Merge pull request #1413 from ViktorTigerstrom/2022-04-default-to-bolt4-tlv-onions
[rust-lightning] / lightning / src / ln / channelmanager.rs
index 8872a905ef347d00ca0a84b76267a995b1c571ff..ae8625d89f4f60eaba2fe8127a5b957a75b5d301 100644 (file)
@@ -34,39 +34,39 @@ use bitcoin::secp256k1::Secp256k1;
 use bitcoin::secp256k1::ecdh::SharedSecret;
 use bitcoin::{LockTime, secp256k1, Sequence};
 
-use chain;
-use chain::{Confirm, ChannelMonitorUpdateErr, Watch, BestBlock};
-use chain::chaininterface::{BroadcasterInterface, ConfirmationTarget, FeeEstimator, LowerBoundedFeeEstimator};
-use chain::channelmonitor::{ChannelMonitor, ChannelMonitorUpdate, ChannelMonitorUpdateStep, HTLC_FAIL_BACK_BUFFER, CLTV_CLAIM_BUFFER, LATENCY_GRACE_PERIOD_BLOCKS, ANTI_REORG_DELAY, MonitorEvent, CLOSED_CHANNEL_UPDATE_ID};
-use chain::transaction::{OutPoint, TransactionData};
+use crate::chain;
+use crate::chain::{Confirm, ChannelMonitorUpdateStatus, Watch, BestBlock};
+use crate::chain::chaininterface::{BroadcasterInterface, ConfirmationTarget, FeeEstimator, LowerBoundedFeeEstimator};
+use crate::chain::channelmonitor::{ChannelMonitor, ChannelMonitorUpdate, ChannelMonitorUpdateStep, HTLC_FAIL_BACK_BUFFER, CLTV_CLAIM_BUFFER, LATENCY_GRACE_PERIOD_BLOCKS, ANTI_REORG_DELAY, MonitorEvent, CLOSED_CHANNEL_UPDATE_ID};
+use crate::chain::transaction::{OutPoint, TransactionData};
 // Since this struct is returned in `list_channels` methods, expose it here in case users want to
 // construct one themselves.
-use ln::{inbound_payment, PaymentHash, PaymentPreimage, PaymentSecret};
-use ln::channel::{Channel, ChannelError, ChannelUpdateStatus, UpdateFulfillCommitFetch};
-use ln::features::{ChannelFeatures, ChannelTypeFeatures, InitFeatures, NodeFeatures};
+use crate::ln::{inbound_payment, PaymentHash, PaymentPreimage, PaymentSecret};
+use crate::ln::channel::{Channel, ChannelError, ChannelUpdateStatus, UpdateFulfillCommitFetch};
+use crate::ln::features::{ChannelFeatures, ChannelTypeFeatures, InitFeatures, NodeFeatures};
 #[cfg(any(feature = "_test_utils", test))]
-use ln::features::InvoiceFeatures;
-use routing::router::{PaymentParameters, Route, RouteHop, RoutePath, RouteParameters};
-use ln::msgs;
-use ln::onion_utils;
-use ln::msgs::{ChannelMessageHandler, DecodeError, LightningError, MAX_VALUE_MSAT};
-use ln::wire::Encode;
-use chain::keysinterface::{Sign, KeysInterface, KeysManager, InMemorySigner, Recipient};
-use util::config::{UserConfig, ChannelConfig};
-use util::events::{EventHandler, EventsProvider, MessageSendEvent, MessageSendEventsProvider, ClosureReason, HTLCDestination};
-use util::{byte_utils, events};
-use util::wakers::{Future, Notifier};
-use util::scid_utils::fake_scid;
-use util::ser::{BigSize, FixedLengthReader, Readable, ReadableArgs, MaybeReadable, Writeable, Writer, VecWriter};
-use util::logger::{Level, Logger};
-use util::errors::APIError;
-
-use io;
-use prelude::*;
+use crate::ln::features::InvoiceFeatures;
+use crate::routing::router::{PaymentParameters, Route, RouteHop, RoutePath, RouteParameters};
+use crate::ln::msgs;
+use crate::ln::onion_utils;
+use crate::ln::msgs::{ChannelMessageHandler, DecodeError, LightningError, MAX_VALUE_MSAT};
+use crate::ln::wire::Encode;
+use crate::chain::keysinterface::{Sign, KeysInterface, KeysManager, Recipient};
+use crate::util::config::{UserConfig, ChannelConfig};
+use crate::util::events::{Event, EventHandler, EventsProvider, MessageSendEvent, MessageSendEventsProvider, ClosureReason, HTLCDestination};
+use crate::util::{byte_utils, events};
+use crate::util::wakers::{Future, Notifier};
+use crate::util::scid_utils::fake_scid;
+use crate::util::ser::{BigSize, FixedLengthReader, Readable, ReadableArgs, MaybeReadable, Writeable, Writer, VecWriter};
+use crate::util::logger::{Level, Logger};
+use crate::util::errors::APIError;
+
+use crate::io;
+use crate::prelude::*;
 use core::{cmp, mem};
 use core::cell::RefCell;
-use io::Read;
-use sync::{Arc, Mutex, MutexGuard, RwLock, RwLockReadGuard};
+use crate::io::Read;
+use crate::sync::{Arc, Mutex, MutexGuard, RwLock, RwLockReadGuard, FairRwLock};
 use core::sync::atomic::{AtomicUsize, Ordering};
 use core::time::Duration;
 use core::ops::Deref;
@@ -112,7 +112,8 @@ pub(super) struct PendingHTLCInfo {
        pub(super) routing: PendingHTLCRouting,
        pub(super) incoming_shared_secret: [u8; 32],
        payment_hash: PaymentHash,
-       pub(super) amt_to_forward: u64,
+       pub(super) incoming_amt_msat: Option<u64>, // Added in 0.0.113
+       pub(super) outgoing_amt_msat: u64,
        pub(super) outgoing_cltv_value: u32,
 }
 
@@ -129,20 +130,22 @@ pub(super) enum PendingHTLCStatus {
        Fail(HTLCFailureMsg),
 }
 
-pub(super) enum HTLCForwardInfo {
-       AddHTLC {
-               forward_info: PendingHTLCInfo,
+pub(super) struct PendingAddHTLCInfo {
+       pub(super) forward_info: PendingHTLCInfo,
 
-               // These fields are produced in `forward_htlcs()` and consumed in
-               // `process_pending_htlc_forwards()` for constructing the
-               // `HTLCSource::PreviousHopData` for failed and forwarded
-               // HTLCs.
-               //
-               // Note that this may be an outbound SCID alias for the associated channel.
-               prev_short_channel_id: u64,
-               prev_htlc_id: u64,
-               prev_funding_outpoint: OutPoint,
-       },
+       // These fields are produced in `forward_htlcs()` and consumed in
+       // `process_pending_htlc_forwards()` for constructing the
+       // `HTLCSource::PreviousHopData` for failed and forwarded
+       // HTLCs.
+       //
+       // Note that this may be an outbound SCID alias for the associated channel.
+       prev_short_channel_id: u64,
+       prev_htlc_id: u64,
+       prev_funding_outpoint: OutPoint,
+}
+
+pub(super) enum HTLCForwardInfo {
+       AddHTLC(PendingAddHTLCInfo),
        FailHTLC {
                htlc_id: u64,
                err_packet: msgs::OnionErrorPacket,
@@ -395,22 +398,6 @@ pub(super) enum RAACommitmentOrder {
 // Note this is only exposed in cfg(test):
 pub(super) struct ChannelHolder<Signer: Sign> {
        pub(super) by_id: HashMap<[u8; 32], Channel<Signer>>,
-       /// SCIDs (and outbound SCID aliases) -> `counterparty_node_id`s and `channel_id`s.
-       ///
-       /// Outbound SCID aliases are added here once the channel is available for normal use, with
-       /// SCIDs being added once the funding transaction is confirmed at the channel's required
-       /// confirmation depth.
-       pub(super) short_to_chan_info: HashMap<u64, (PublicKey, [u8; 32])>,
-       /// SCID/SCID Alias -> forward infos. Key of 0 means payments received.
-       ///
-       /// Note that because we may have an SCID Alias as the key we can have two entries per channel,
-       /// though in practice we probably won't be receiving HTLCs for a channel both via the alias
-       /// and via the classic SCID.
-       ///
-       /// Note that while this is held in the same mutex as the channels themselves, no consistency
-       /// guarantees are made about the existence of a channel with the short id here, nor the short
-       /// ids in the PendingHTLCInfo!
-       pub(super) forward_htlcs: HashMap<u64, Vec<HTLCForwardInfo>>,
        /// Map from payment hash to the payment data and any HTLCs which are to us and can be
        /// failed/claimed by the user.
        ///
@@ -483,6 +470,7 @@ pub(crate) enum PendingOutboundPayment {
        Fulfilled {
                session_privs: HashSet<[u8; 32]>,
                payment_hash: Option<PaymentHash>,
+               timer_ticks_without_htlcs: u8,
        },
        /// When a payer gives up trying to retry a payment, they inform us, letting us generate a
        /// `PaymentFailed` event when all HTLCs have irrevocably failed. This avoids a number of race
@@ -498,12 +486,6 @@ pub(crate) enum PendingOutboundPayment {
 }
 
 impl PendingOutboundPayment {
-       fn is_retryable(&self) -> bool {
-               match self {
-                       PendingOutboundPayment::Retryable { .. } => true,
-                       _ => false,
-               }
-       }
        fn is_fulfilled(&self) -> bool {
                match self {
                        PendingOutboundPayment::Fulfilled { .. } => true,
@@ -542,7 +524,7 @@ impl PendingOutboundPayment {
                                => session_privs,
                });
                let payment_hash = self.payment_hash();
-               *self = PendingOutboundPayment::Fulfilled { session_privs, payment_hash };
+               *self = PendingOutboundPayment::Fulfilled { session_privs, payment_hash, timer_ticks_without_htlcs: 0 };
        }
 
        fn mark_abandoned(&mut self) -> Result<(), ()> {
@@ -627,7 +609,7 @@ impl PendingOutboundPayment {
 /// concrete type of the KeysManager.
 ///
 /// (C-not exported) as Arcs don't make sense in bindings
-pub type SimpleArcChannelManager<M, T, F, L> = ChannelManager<InMemorySigner, Arc<M>, Arc<T>, Arc<KeysManager>, Arc<F>, Arc<L>>;
+pub type SimpleArcChannelManager<M, T, F, L> = ChannelManager<Arc<M>, Arc<T>, Arc<KeysManager>, Arc<F>, Arc<L>>;
 
 /// SimpleRefChannelManager is a type alias for a ChannelManager reference, and is the reference
 /// counterpart to the SimpleArcChannelManager type alias. Use this type by default when you don't
@@ -639,7 +621,7 @@ pub type SimpleArcChannelManager<M, T, F, L> = ChannelManager<InMemorySigner, Ar
 /// concrete type of the KeysManager.
 ///
 /// (C-not exported) as Arcs don't make sense in bindings
-pub type SimpleRefChannelManager<'a, 'b, 'c, 'd, 'e, M, T, F, L> = ChannelManager<InMemorySigner, &'a M, &'b T, &'c KeysManager, &'d F, &'e L>;
+pub type SimpleRefChannelManager<'a, 'b, 'c, 'd, 'e, M, T, F, L> = ChannelManager<&'a M, &'b T, &'c KeysManager, &'d F, &'e L>;
 
 /// Manager which keeps track of a number of channels and sends messages to the appropriate
 /// channel, also tracking HTLC preimages and forwarding onion packets appropriately.
@@ -677,10 +659,44 @@ pub type SimpleRefChannelManager<'a, 'b, 'c, 'd, 'e, M, T, F, L> = ChannelManage
 /// essentially you should default to using a SimpleRefChannelManager, and use a
 /// SimpleArcChannelManager when you require a ChannelManager with a static lifetime, such as when
 /// you're using lightning-net-tokio.
-pub struct ChannelManager<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
-       where M::Target: chain::Watch<Signer>,
+//
+// Lock order:
+// The tree structure below illustrates the lock order requirements for the different locks of the
+// `ChannelManager`. Locks can be held at the same time if they are on the same branch in the tree,
+// and should then be taken in the order of the lowest to the highest level in the tree.
+// Note that locks on different branches shall not be taken at the same time, as doing so will
+// create a new lock order for those specific locks in the order they were taken.
+//
+// Lock order tree:
+//
+// `total_consistency_lock`
+//  |
+//  |__`forward_htlcs`
+//  |
+//  |__`channel_state`
+//  |   |
+//  |   |__`id_to_peer`
+//  |   |
+//  |   |__`short_to_chan_info`
+//  |   |
+//  |   |__`per_peer_state`
+//  |       |
+//  |       |__`outbound_scid_aliases`
+//  |       |
+//  |       |__`pending_inbound_payments`
+//  |           |
+//  |           |__`pending_outbound_payments`
+//  |               |
+//  |               |__`best_block`
+//  |               |
+//  |               |__`pending_events`
+//  |                   |
+//  |                   |__`pending_background_events`
+//
+pub struct ChannelManager<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
                                L::Target: Logger,
 {
@@ -690,22 +706,25 @@ pub struct ChannelManager<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref,
        chain_monitor: M,
        tx_broadcaster: T,
 
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        #[cfg(test)]
        pub(super) best_block: RwLock<BestBlock>,
        #[cfg(not(test))]
        best_block: RwLock<BestBlock>,
        secp_ctx: Secp256k1<secp256k1::All>,
 
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        #[cfg(any(test, feature = "_test_utils"))]
-       pub(super) channel_state: Mutex<ChannelHolder<Signer>>,
+       pub(super) channel_state: Mutex<ChannelHolder<<K::Target as KeysInterface>::Signer>>,
        #[cfg(not(any(test, feature = "_test_utils")))]
-       channel_state: Mutex<ChannelHolder<Signer>>,
+       channel_state: Mutex<ChannelHolder<<K::Target as KeysInterface>::Signer>>,
 
        /// Storage for PaymentSecrets and any requirements on future inbound payments before we will
        /// expose them to users via a PaymentReceived event. HTLCs which do not meet the requirements
        /// here are failed when we process them as pending-forwardable-HTLCs, and entries are removed
        /// after we generate a PaymentReceived upon receipt of all MPP parts or when they time out.
-       /// Locked *after* channel_state.
+       ///
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        pending_inbound_payments: Mutex<HashMap<PaymentHash, PendingInboundPayment>>,
 
        /// The session_priv bytes and retry metadata of outbound payments which are pending resolution.
@@ -719,13 +738,30 @@ pub struct ChannelManager<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref,
        ///
        /// See `PendingOutboundPayment` documentation for more info.
        ///
-       /// Locked *after* channel_state.
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        pending_outbound_payments: Mutex<HashMap<PaymentId, PendingOutboundPayment>>,
 
+       /// SCID/SCID Alias -> forward infos. Key of 0 means payments received.
+       ///
+       /// Note that because we may have an SCID Alias as the key we can have two entries per channel,
+       /// though in practice we probably won't be receiving HTLCs for a channel both via the alias
+       /// and via the classic SCID.
+       ///
+       /// Note that no consistency guarantees are made about the existence of a channel with the
+       /// `short_channel_id` here, nor the `short_channel_id` in the `PendingHTLCInfo`!
+       ///
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
+       #[cfg(test)]
+       pub(super) forward_htlcs: Mutex<HashMap<u64, Vec<HTLCForwardInfo>>>,
+       #[cfg(not(test))]
+       forward_htlcs: Mutex<HashMap<u64, Vec<HTLCForwardInfo>>>,
+
        /// The set of outbound SCID aliases across all our channels, including unconfirmed channels
        /// and some closed channels which reached a usable state prior to being closed. This is used
        /// only to avoid duplicates, and is not persisted explicitly to disk, but rebuilt from the
        /// active channel list on load.
+       ///
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        outbound_scid_aliases: Mutex<HashSet<u64>>,
 
        /// `channel_id` -> `counterparty_node_id`.
@@ -745,8 +781,26 @@ pub struct ChannelManager<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref,
        /// We should add `counterparty_node_id`s to `MonitorEvent`s, and eventually rely on it in the
        /// future. That would make this map redundant, as only the `ChannelManager::per_peer_state` is
        /// required to access the channel with the `counterparty_node_id`.
+       ///
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        id_to_peer: Mutex<HashMap<[u8; 32], PublicKey>>,
 
+       /// SCIDs (and outbound SCID aliases) -> `counterparty_node_id`s and `channel_id`s.
+       ///
+       /// Outbound SCID aliases are added here once the channel is available for normal use, with
+       /// SCIDs being added once the funding transaction is confirmed at the channel's required
+       /// confirmation depth.
+       ///
+       /// Note that while this holds `counterparty_node_id`s and `channel_id`s, no consistency
+       /// guarantees are made about the existence of a peer with the `counterparty_node_id` nor a
+       /// channel with the `channel_id` in our other maps.
+       ///
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
+       #[cfg(test)]
+       pub(super) short_to_chan_info: FairRwLock<HashMap<u64, (PublicKey, [u8; 32])>>,
+       #[cfg(not(test))]
+       short_to_chan_info: FairRwLock<HashMap<u64, (PublicKey, [u8; 32])>>,
+
        our_network_key: SecretKey,
        our_network_pubkey: PublicKey,
 
@@ -776,10 +830,12 @@ pub struct ChannelManager<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref,
        /// operate on the inner value freely. Sadly, this prevents parallel operation when opening a
        /// new channel.
        ///
-       /// If also holding `channel_state` lock, must lock `channel_state` prior to `per_peer_state`.
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        per_peer_state: RwLock<HashMap<PublicKey, Mutex<PeerState>>>,
 
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        pending_events: Mutex<Vec<events::Event>>,
+       /// See `ChannelManager` struct-level documentation for lock order requirements.
        pending_background_events: Mutex<Vec<BackgroundEvent>>,
        /// Used when we have to take a BIG lock to make sure everything is self-consistent.
        /// Essentially just when we're serializing ourselves out.
@@ -913,13 +969,14 @@ const CHECK_CLTV_EXPIRY_SANITY: u32 = MIN_CLTV_EXPIRY_DELTA as u32 - LATENCY_GRA
 #[allow(dead_code)]
 const CHECK_CLTV_EXPIRY_SANITY_2: u32 = MIN_CLTV_EXPIRY_DELTA as u32 - LATENCY_GRACE_PERIOD_BLOCKS - 2*CLTV_CLAIM_BUFFER;
 
-/// The number of blocks before we consider an outbound payment for expiry if it doesn't have any
-/// pending HTLCs in flight.
-pub(crate) const PAYMENT_EXPIRY_BLOCKS: u32 = 3;
-
 /// The number of ticks of [`ChannelManager::timer_tick_occurred`] until expiry of incomplete MPPs
 pub(crate) const MPP_TIMEOUT_TICKS: u8 = 3;
 
+/// The number of ticks of [`ChannelManager::timer_tick_occurred`] until we time-out the
+/// idempotency of payments by [`PaymentId`]. See
+/// [`ChannelManager::remove_stale_resolved_payments`].
+pub(crate) const IDEMPOTENCY_TIMEOUT_TICKS: u8 = 7;
+
 /// Information needed for constructing an invoice route hint for this channel.
 #[derive(Clone, Debug, PartialEq)]
 pub struct CounterpartyForwardingInfo {
@@ -1160,19 +1217,22 @@ pub enum PaymentSendFailure {
        /// All paths which were attempted failed to send, with no channel state change taking place.
        /// You can freely retry the payment in full (though you probably want to do so over different
        /// paths than the ones selected).
+       ///
+       /// [`ChannelManager::abandon_payment`] does *not* need to be called for this payment and
+       /// [`ChannelManager::retry_payment`] will *not* work for this payment.
        AllFailedRetrySafe(Vec<APIError>),
        /// Some paths which were attempted failed to send, though possibly not all. At least some
        /// paths have irrevocably committed to the HTLC and retrying the payment in full would result
        /// in over-/re-payment.
        ///
        /// The results here are ordered the same as the paths in the route object which was passed to
-       /// send_payment, and any Errs which are not APIError::MonitorUpdateFailed can be safely
-       /// retried (though there is currently no API with which to do so).
+       /// send_payment, and any `Err`s which are not [`APIError::MonitorUpdateInProgress`] can be
+       /// safely retried via [`ChannelManager::retry_payment`].
        ///
-       /// Any entries which contain Err(APIError::MonitorUpdateFailed) or Ok(()) MUST NOT be retried
-       /// as they will result in over-/re-payment. These HTLCs all either successfully sent (in the
-       /// case of Ok(())) or will send once a [`MonitorEvent::UpdateCompleted`] is provided for the
-       /// next-hop channel with the latest update_id.
+       /// Any entries which contain `Err(APIError::MonitorUpdateInprogress)` or `Ok(())` MUST NOT be
+       /// retried as they will result in over-/re-payment. These HTLCs all either successfully sent
+       /// (in the case of `Ok(())`) or will send once a [`MonitorEvent::Completed`] is provided for
+       /// the next-hop channel with the latest update_id.
        PartialFailure {
                /// The errors themselves, in the same order as the route hops.
                results: Vec<Result<(), APIError>>,
@@ -1250,9 +1310,11 @@ macro_rules! handle_error {
 }
 
 macro_rules! update_maps_on_chan_removal {
-       ($self: expr, $short_to_chan_info: expr, $channel: expr) => {
+       ($self: expr, $channel: expr) => {{
+               $self.id_to_peer.lock().unwrap().remove(&$channel.channel_id());
+               let mut short_to_chan_info = $self.short_to_chan_info.write().unwrap();
                if let Some(short_id) = $channel.get_short_channel_id() {
-                       $short_to_chan_info.remove(&short_id);
+                       short_to_chan_info.remove(&short_id);
                } else {
                        // If the channel was never confirmed on-chain prior to its closure, remove the
                        // outbound SCID alias we used for it from the collision-prevention set. While we
@@ -1263,14 +1325,13 @@ macro_rules! update_maps_on_chan_removal {
                        let alias_removed = $self.outbound_scid_aliases.lock().unwrap().remove(&$channel.outbound_scid_alias());
                        debug_assert!(alias_removed);
                }
-               $self.id_to_peer.lock().unwrap().remove(&$channel.channel_id());
-               $short_to_chan_info.remove(&$channel.outbound_scid_alias());
-       }
+               short_to_chan_info.remove(&$channel.outbound_scid_alias());
+       }}
 }
 
 /// Returns (boolean indicating if we should remove the Channel object from memory, a mapped error)
 macro_rules! convert_chan_err {
-       ($self: ident, $err: expr, $short_to_chan_info: expr, $channel: expr, $channel_id: expr) => {
+       ($self: ident, $err: expr, $channel: expr, $channel_id: expr) => {
                match $err {
                        ChannelError::Warn(msg) => {
                                (false, MsgHandleErrInternal::from_chan_no_close(ChannelError::Warn(msg), $channel_id.clone()))
@@ -1280,7 +1341,7 @@ macro_rules! convert_chan_err {
                        },
                        ChannelError::Close(msg) => {
                                log_error!($self.logger, "Closing channel {} due to close-required error: {}", log_bytes!($channel_id[..]), msg);
-                               update_maps_on_chan_removal!($self, $short_to_chan_info, $channel);
+                               update_maps_on_chan_removal!($self, $channel);
                                let shutdown_res = $channel.force_shutdown(true);
                                (true, MsgHandleErrInternal::from_finish_shutdown(msg, *$channel_id, $channel.get_user_id(),
                                        shutdown_res, $self.get_channel_update_for_broadcast(&$channel).ok()))
@@ -1290,11 +1351,11 @@ macro_rules! convert_chan_err {
 }
 
 macro_rules! break_chan_entry {
-       ($self: ident, $res: expr, $channel_state: expr, $entry: expr) => {
+       ($self: ident, $res: expr, $entry: expr) => {
                match $res {
                        Ok(res) => res,
                        Err(e) => {
-                               let (drop, res) = convert_chan_err!($self, e, $channel_state.short_to_chan_info, $entry.get_mut(), $entry.key());
+                               let (drop, res) = convert_chan_err!($self, e, $entry.get_mut(), $entry.key());
                                if drop {
                                        $entry.remove_entry();
                                }
@@ -1305,11 +1366,11 @@ macro_rules! break_chan_entry {
 }
 
 macro_rules! try_chan_entry {
-       ($self: ident, $res: expr, $channel_state: expr, $entry: expr) => {
+       ($self: ident, $res: expr, $entry: expr) => {
                match $res {
                        Ok(res) => res,
                        Err(e) => {
-                               let (drop, res) = convert_chan_err!($self, e, $channel_state.short_to_chan_info, $entry.get_mut(), $entry.key());
+                               let (drop, res) = convert_chan_err!($self, e, $entry.get_mut(), $entry.key());
                                if drop {
                                        $entry.remove_entry();
                                }
@@ -1320,21 +1381,21 @@ macro_rules! try_chan_entry {
 }
 
 macro_rules! remove_channel {
-       ($self: expr, $channel_state: expr, $entry: expr) => {
+       ($self: expr, $entry: expr) => {
                {
                        let channel = $entry.remove_entry().1;
-                       update_maps_on_chan_removal!($self, $channel_state.short_to_chan_info, channel);
+                       update_maps_on_chan_removal!($self, channel);
                        channel
                }
        }
 }
 
-macro_rules! handle_monitor_err {
-       ($self: ident, $err: expr, $short_to_chan_info: expr, $chan: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $resend_channel_ready: expr, $failed_forwards: expr, $failed_fails: expr, $failed_finalized_fulfills: expr, $chan_id: expr) => {
+macro_rules! handle_monitor_update_res {
+       ($self: ident, $err: expr, $chan: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $resend_channel_ready: expr, $failed_forwards: expr, $failed_fails: expr, $failed_finalized_fulfills: expr, $chan_id: expr) => {
                match $err {
-                       ChannelMonitorUpdateErr::PermanentFailure => {
-                               log_error!($self.logger, "Closing channel {} due to monitor update ChannelMonitorUpdateErr::PermanentFailure", log_bytes!($chan_id[..]));
-                               update_maps_on_chan_removal!($self, $short_to_chan_info, $chan);
+                       ChannelMonitorUpdateStatus::PermanentFailure => {
+                               log_error!($self.logger, "Closing channel {} due to monitor update ChannelMonitorUpdateStatus::PermanentFailure", log_bytes!($chan_id[..]));
+                               update_maps_on_chan_removal!($self, $chan);
                                // TODO: $failed_fails is dropped here, which will cause other channels to hit the
                                // chain in a confused state! We need to move them into the ChannelMonitor which
                                // will be responsible for failing backwards once things confirm on-chain.
@@ -1348,8 +1409,8 @@ macro_rules! handle_monitor_err {
                                                $chan.force_shutdown(false), $self.get_channel_update_for_broadcast(&$chan).ok() ));
                                (res, true)
                        },
-                       ChannelMonitorUpdateErr::TemporaryFailure => {
-                               log_info!($self.logger, "Disabling channel {} due to monitor update TemporaryFailure. On restore will send {} and process {} forwards, {} fails, and {} fulfill finalizations",
+                       ChannelMonitorUpdateStatus::InProgress => {
+                               log_info!($self.logger, "Disabling channel {} due to monitor update in progress. On restore will send {} and process {} forwards, {} fails, and {} fulfill finalizations",
                                                log_bytes!($chan_id[..]),
                                                if $resend_commitment && $resend_raa {
                                                                match $action_type {
@@ -1368,73 +1429,73 @@ macro_rules! handle_monitor_err {
                                if !$resend_raa {
                                        debug_assert!($action_type == RAACommitmentOrder::CommitmentFirst || !$resend_commitment);
                                }
-                               $chan.monitor_update_failed($resend_raa, $resend_commitment, $resend_channel_ready, $failed_forwards, $failed_fails, $failed_finalized_fulfills);
+                               $chan.monitor_updating_paused($resend_raa, $resend_commitment, $resend_channel_ready, $failed_forwards, $failed_fails, $failed_finalized_fulfills);
                                (Err(MsgHandleErrInternal::from_chan_no_close(ChannelError::Ignore("Failed to update ChannelMonitor".to_owned()), *$chan_id)), false)
                        },
+                       ChannelMonitorUpdateStatus::Completed => {
+                               (Ok(()), false)
+                       },
                }
        };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $resend_channel_ready: expr, $failed_forwards: expr, $failed_fails: expr, $failed_finalized_fulfills: expr) => { {
-               let (res, drop) = handle_monitor_err!($self, $err, $channel_state.short_to_chan_info, $entry.get_mut(), $action_type, $resend_raa, $resend_commitment, $resend_channel_ready, $failed_forwards, $failed_fails, $failed_finalized_fulfills, $entry.key());
+       ($self: ident, $err: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $resend_channel_ready: expr, $failed_forwards: expr, $failed_fails: expr, $failed_finalized_fulfills: expr) => { {
+               let (res, drop) = handle_monitor_update_res!($self, $err, $entry.get_mut(), $action_type, $resend_raa, $resend_commitment, $resend_channel_ready, $failed_forwards, $failed_fails, $failed_finalized_fulfills, $entry.key());
                if drop {
                        $entry.remove_entry();
                }
                res
        } };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $chan_id: expr, COMMITMENT_UPDATE_ONLY) => { {
+       ($self: ident, $err: expr, $entry: expr, $action_type: path, $chan_id: expr, COMMITMENT_UPDATE_ONLY) => { {
                debug_assert!($action_type == RAACommitmentOrder::CommitmentFirst);
-               handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, false, true, false, Vec::new(), Vec::new(), Vec::new(), $chan_id)
+               handle_monitor_update_res!($self, $err, $entry, $action_type, false, true, false, Vec::new(), Vec::new(), Vec::new(), $chan_id)
        } };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $chan_id: expr, NO_UPDATE) => {
-               handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, false, false, false, Vec::new(), Vec::new(), Vec::new(), $chan_id)
+       ($self: ident, $err: expr, $entry: expr, $action_type: path, $chan_id: expr, NO_UPDATE) => {
+               handle_monitor_update_res!($self, $err, $entry, $action_type, false, false, false, Vec::new(), Vec::new(), Vec::new(), $chan_id)
        };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_channel_ready: expr, OPTIONALLY_RESEND_FUNDING_LOCKED) => {
-               handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, false, false, $resend_channel_ready, Vec::new(), Vec::new(), Vec::new())
+       ($self: ident, $err: expr, $entry: expr, $action_type: path, $resend_channel_ready: expr, OPTIONALLY_RESEND_FUNDING_LOCKED) => {
+               handle_monitor_update_res!($self, $err, $entry, $action_type, false, false, $resend_channel_ready, Vec::new(), Vec::new(), Vec::new())
        };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr) => {
-               handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, $resend_raa, $resend_commitment, false, Vec::new(), Vec::new(), Vec::new())
+       ($self: ident, $err: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr) => {
+               handle_monitor_update_res!($self, $err, $entry, $action_type, $resend_raa, $resend_commitment, false, Vec::new(), Vec::new(), Vec::new())
        };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $failed_forwards: expr, $failed_fails: expr) => {
-               handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, $resend_raa, $resend_commitment, false, $failed_forwards, $failed_fails, Vec::new())
+       ($self: ident, $err: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $failed_forwards: expr, $failed_fails: expr) => {
+               handle_monitor_update_res!($self, $err, $entry, $action_type, $resend_raa, $resend_commitment, false, $failed_forwards, $failed_fails, Vec::new())
        };
 }
 
-macro_rules! return_monitor_err {
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr) => {
-               return handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, $resend_raa, $resend_commitment);
-       };
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr, $failed_forwards: expr, $failed_fails: expr) => {
-               return handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, $resend_raa, $resend_commitment, $failed_forwards, $failed_fails);
-       }
-}
-
-// Does not break in case of TemporaryFailure!
-macro_rules! maybe_break_monitor_err {
-       ($self: ident, $err: expr, $channel_state: expr, $entry: expr, $action_type: path, $resend_raa: expr, $resend_commitment: expr) => {
-               match (handle_monitor_err!($self, $err, $channel_state, $entry, $action_type, $resend_raa, $resend_commitment), $err) {
-                       (e, ChannelMonitorUpdateErr::PermanentFailure) => {
-                               break e;
-                       },
-                       (_, ChannelMonitorUpdateErr::TemporaryFailure) => { },
-               }
-       }
-}
-
 macro_rules! send_channel_ready {
-       ($short_to_chan_info: expr, $pending_msg_events: expr, $channel: expr, $channel_ready_msg: expr) => {
+       ($self: ident, $pending_msg_events: expr, $channel: expr, $channel_ready_msg: expr) => {{
                $pending_msg_events.push(events::MessageSendEvent::SendChannelReady {
                        node_id: $channel.get_counterparty_node_id(),
                        msg: $channel_ready_msg,
                });
                // Note that we may send a `channel_ready` multiple times for a channel if we reconnect, so
                // we allow collisions, but we shouldn't ever be updating the channel ID pointed to.
-               let outbound_alias_insert = $short_to_chan_info.insert($channel.outbound_scid_alias(), ($channel.get_counterparty_node_id(), $channel.channel_id()));
+               let mut short_to_chan_info = $self.short_to_chan_info.write().unwrap();
+               let outbound_alias_insert = short_to_chan_info.insert($channel.outbound_scid_alias(), ($channel.get_counterparty_node_id(), $channel.channel_id()));
                assert!(outbound_alias_insert.is_none() || outbound_alias_insert.unwrap() == ($channel.get_counterparty_node_id(), $channel.channel_id()),
                        "SCIDs should never collide - ensure you weren't behind the chain tip by a full month when creating channels");
                if let Some(real_scid) = $channel.get_short_channel_id() {
-                       let scid_insert = $short_to_chan_info.insert(real_scid, ($channel.get_counterparty_node_id(), $channel.channel_id()));
+                       let scid_insert = short_to_chan_info.insert(real_scid, ($channel.get_counterparty_node_id(), $channel.channel_id()));
                        assert!(scid_insert.is_none() || scid_insert.unwrap() == ($channel.get_counterparty_node_id(), $channel.channel_id()),
                                "SCIDs should never collide - ensure you weren't behind the chain tip by a full month when creating channels");
                }
+       }}
+}
+
+macro_rules! emit_channel_ready_event {
+       ($self: expr, $channel: expr) => {
+               if $channel.should_emit_channel_ready_event() {
+                       {
+                               let mut pending_events = $self.pending_events.lock().unwrap();
+                               pending_events.push(events::Event::ChannelReady {
+                                       channel_id: $channel.channel_id(),
+                                       user_channel_id: $channel.get_user_id(),
+                                       counterparty_node_id: $channel.get_counterparty_node_id(),
+                                       channel_type: $channel.get_channel_type().clone(),
+                               });
+                       }
+                       $channel.set_channel_ready_event_emitted();
+               }
        }
 }
 
@@ -1472,7 +1533,7 @@ macro_rules! handle_chan_restoration_locked {
                                // Similar to the above, this implies that we're letting the channel_ready fly
                                // before it should be allowed to.
                                assert!(chanmon_update.is_none());
-                               send_channel_ready!($channel_state.short_to_chan_info, $channel_state.pending_msg_events, $channel_entry.get(), msg);
+                               send_channel_ready!($self, $channel_state.pending_msg_events, $channel_entry.get(), msg);
                        }
                        if let Some(msg) = $announcement_sigs {
                                $channel_state.pending_msg_events.push(events::MessageSendEvent::SendAnnouncementSignatures {
@@ -1481,6 +1542,8 @@ macro_rules! handle_chan_restoration_locked {
                                });
                        }
 
+                       emit_channel_ready_event!($self, $channel_entry.get_mut());
+
                        let funding_broadcastable: Option<Transaction> = $funding_broadcastable; // Force type-checking to resolve
                        if let Some(monitor_update) = chanmon_update {
                                // We only ever broadcast a funding transaction in response to a funding_signed
@@ -1493,15 +1556,18 @@ macro_rules! handle_chan_restoration_locked {
                                // only case where we can get a new ChannelMonitorUpdate would be if we also
                                // have some commitment updates to send as well.
                                assert!($commitment_update.is_some());
-                               if let Err(e) = $self.chain_monitor.update_channel($channel_entry.get().get_funding_txo().unwrap(), monitor_update) {
-                                       // channel_reestablish doesn't guarantee the order it returns is sensical
-                                       // for the messages it returns, but if we're setting what messages to
-                                       // re-transmit on monitor update success, we need to make sure it is sane.
-                                       let mut order = $order;
-                                       if $raa.is_none() {
-                                               order = RAACommitmentOrder::CommitmentFirst;
+                               match $self.chain_monitor.update_channel($channel_entry.get().get_funding_txo().unwrap(), monitor_update) {
+                                       ChannelMonitorUpdateStatus::Completed => {},
+                                       e => {
+                                               // channel_reestablish doesn't guarantee the order it returns is sensical
+                                               // for the messages it returns, but if we're setting what messages to
+                                               // re-transmit on monitor update success, we need to make sure it is sane.
+                                               let mut order = $order;
+                                               if $raa.is_none() {
+                                                       order = RAACommitmentOrder::CommitmentFirst;
+                                               }
+                                               break handle_monitor_update_res!($self, e, $channel_entry, order, $raa.is_some(), true);
                                        }
-                                       break handle_monitor_err!($self, e, $channel_state, $channel_entry, order, $raa.is_some(), true);
                                }
                        }
 
@@ -1561,10 +1627,10 @@ macro_rules! post_handle_chan_restoration {
        } }
 }
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelManager<Signer, M, T, K, F, L>
-       where M::Target: chain::Watch<Signer>,
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelManager<M, T, K, F, L>
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
         L::Target: Logger,
 {
@@ -1594,15 +1660,15 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                        channel_state: Mutex::new(ChannelHolder{
                                by_id: HashMap::new(),
-                               short_to_chan_info: HashMap::new(),
-                               forward_htlcs: HashMap::new(),
                                claimable_htlcs: HashMap::new(),
                                pending_msg_events: Vec::new(),
                        }),
                        outbound_scid_aliases: Mutex::new(HashSet::new()),
                        pending_inbound_payments: Mutex::new(HashMap::new()),
                        pending_outbound_payments: Mutex::new(HashMap::new()),
+                       forward_htlcs: Mutex::new(HashMap::new()),
                        id_to_peer: Mutex::new(HashMap::new()),
+                       short_to_chan_info: FairRwLock::new(HashMap::new()),
 
                        our_network_key: keys_manager.get_node_secret(Recipient::Node).unwrap(),
                        our_network_pubkey: PublicKey::from_secret_key(&secp_ctx, &keys_manager.get_node_secret(Recipient::Node).unwrap()),
@@ -1730,7 +1796,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                Ok(temporary_channel_id)
        }
 
-       fn list_channels_with_filter<Fn: FnMut(&(&[u8; 32], &Channel<Signer>)) -> bool>(&self, f: Fn) -> Vec<ChannelDetails> {
+       fn list_channels_with_filter<Fn: FnMut(&(&[u8; 32], &Channel<<K::Target as KeysInterface>::Signer>)) -> bool>(&self, f: Fn) -> Vec<ChannelDetails> {
                let mut res = Vec::new();
                {
                        let channel_state = self.channel_state.lock().unwrap();
@@ -1812,7 +1878,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        }
 
        /// Helper function that issues the channel close events
-       fn issue_channel_close_events(&self, channel: &Channel<Signer>, closure_reason: ClosureReason) {
+       fn issue_channel_close_events(&self, channel: &Channel<<K::Target as KeysInterface>::Signer>, closure_reason: ClosureReason) {
                let mut pending_events_lock = self.pending_events.lock().unwrap();
                match channel.unbroadcasted_funding() {
                        Some(transaction) => {
@@ -1852,13 +1918,12 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                                        // Update the monitor with the shutdown script if necessary.
                                        if let Some(monitor_update) = monitor_update {
-                                               if let Err(e) = self.chain_monitor.update_channel(chan_entry.get().get_funding_txo().unwrap(), monitor_update) {
-                                                       let (result, is_permanent) =
-                                                               handle_monitor_err!(self, e, channel_state.short_to_chan_info, chan_entry.get_mut(), RAACommitmentOrder::CommitmentFirst, chan_entry.key(), NO_UPDATE);
-                                                       if is_permanent {
-                                                               remove_channel!(self, channel_state, chan_entry);
-                                                               break result;
-                                                       }
+                                               let update_res = self.chain_monitor.update_channel(chan_entry.get().get_funding_txo().unwrap(), monitor_update);
+                                               let (result, is_permanent) =
+                                                       handle_monitor_update_res!(self, update_res, chan_entry.get_mut(), RAACommitmentOrder::CommitmentFirst, chan_entry.key(), NO_UPDATE);
+                                               if is_permanent {
+                                                       remove_channel!(self, chan_entry);
+                                                       break result;
                                                }
                                        }
 
@@ -1868,7 +1933,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        });
 
                                        if chan_entry.get().is_shutdown() {
-                                               let channel = remove_channel!(self, channel_state, chan_entry);
+                                               let channel = remove_channel!(self, chan_entry);
                                                if let Ok(channel_update) = self.get_channel_update_for_broadcast(&channel) {
                                                        channel_state.pending_msg_events.push(events::MessageSendEvent::BroadcastChannelUpdate {
                                                                msg: channel_update
@@ -1884,7 +1949,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                for htlc_source in failed_htlcs.drain(..) {
                        let receiver = HTLCDestination::NextHopChannel { node_id: Some(*counterparty_node_id), channel_id: *channel_id };
-                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), htlc_source.0, &htlc_source.1, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
+                       self.fail_htlc_backwards_internal(htlc_source.0, &htlc_source.1, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
                }
 
                let _ = handle_error!(self, result, *counterparty_node_id);
@@ -1941,8 +2006,8 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                log_debug!(self.logger, "Finishing force-closure of channel with {} HTLCs to fail", failed_htlcs.len());
                for htlc_source in failed_htlcs.drain(..) {
                        let (source, payment_hash, counterparty_node_id, channel_id) = htlc_source;
-                       let receiver = HTLCDestination::NextHopChannel { node_id: Some(counterparty_node_id), channel_id: channel_id };
-                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), source, &payment_hash, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
+                       let receiver = HTLCDestination::NextHopChannel { node_id: Some(counterparty_node_id), channel_id };
+                       self.fail_htlc_backwards_internal(source, &payment_hash, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
                }
                if let Some((funding_txo, monitor_update)) = monitor_update_option {
                        // There isn't anything we can do if we get an update failure - we're already
@@ -1969,7 +2034,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                } else {
                                        self.issue_channel_close_events(chan.get(),ClosureReason::HolderForceClosed);
                                }
-                               remove_channel!(self, channel_state, chan)
+                               remove_channel!(self, chan)
                        } else {
                                return Err(APIError::ChannelUnavailable{err: "No such channel".to_owned()});
                        }
@@ -2073,13 +2138,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                }
 
                let routing = match hop_data.format {
-                       msgs::OnionHopDataFormat::Legacy { .. } => {
-                               return Err(ReceiveError {
-                                       err_code: 0x4000|0x2000|3,
-                                       err_data: Vec::new(),
-                                       msg: "We require payment_secrets",
-                               });
-                       },
                        msgs::OnionHopDataFormat::NonFinalNode { .. } => {
                                return Err(ReceiveError {
                                        err_code: 0x4000|22,
@@ -2132,7 +2190,8 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        routing,
                        payment_hash,
                        incoming_shared_secret: shared_secret,
-                       amt_to_forward: amt_msat,
+                       incoming_amt_msat: Some(amt_msat),
+                       outgoing_amt_msat: amt_msat,
                        outgoing_cltv_value: hop_data.outgoing_cltv_value,
                })
        }
@@ -2214,7 +2273,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                };
 
                                let short_channel_id = match next_hop_data.format {
-                                       msgs::OnionHopDataFormat::Legacy { short_channel_id } => short_channel_id,
                                        msgs::OnionHopDataFormat::NonFinalNode { short_channel_id } => short_channel_id,
                                        msgs::OnionHopDataFormat::FinalNode { .. } => {
                                                return_err!("Final Node OnionHopData provided for us as an intermediary node", 0x4000 | 22, &[0;0]);
@@ -2228,25 +2286,26 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        },
                                        payment_hash: msg.payment_hash.clone(),
                                        incoming_shared_secret: shared_secret,
-                                       amt_to_forward: next_hop_data.amt_to_forward,
+                                       incoming_amt_msat: Some(msg.amount_msat),
+                                       outgoing_amt_msat: next_hop_data.amt_to_forward,
                                        outgoing_cltv_value: next_hop_data.outgoing_cltv_value,
                                })
                        }
                };
 
-               if let &PendingHTLCStatus::Forward(PendingHTLCInfo { ref routing, ref amt_to_forward, ref outgoing_cltv_value, .. }) = &pending_forward_info {
+               if let &PendingHTLCStatus::Forward(PendingHTLCInfo { ref routing, ref outgoing_amt_msat, ref outgoing_cltv_value, .. }) = &pending_forward_info {
                        // If short_channel_id is 0 here, we'll reject the HTLC as there cannot be a channel
                        // with a short_channel_id of 0. This is important as various things later assume
                        // short_channel_id is non-0 in any ::Forward.
                        if let &PendingHTLCRouting::Forward { ref short_channel_id, .. } = routing {
                                if let Some((err, code, chan_update)) = loop {
+                                       let id_option = self.short_to_chan_info.read().unwrap().get(&short_channel_id).cloned();
                                        let mut channel_state = self.channel_state.lock().unwrap();
-                                       let id_option = channel_state.short_to_chan_info.get(&short_channel_id).cloned();
                                        let forwarding_id_opt = match id_option {
                                                None => { // unknown_next_peer
                                                        // Note that this is likely a timing oracle for detecting whether an scid is a
                                                        // phantom.
-                                                       if fake_scid::is_valid_phantom(&self.fake_scid_rand_bytes, *short_channel_id) {
+                                                       if fake_scid::is_valid_phantom(&self.fake_scid_rand_bytes, *short_channel_id, &self.genesis_hash) {
                                                                None
                                                        } else {
                                                                break Some(("Don't have available channel for forwarding as requested.", 0x4000 | 10, None));
@@ -2255,7 +2314,14 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                Some((_cp_id, chan_id)) => Some(chan_id.clone()),
                                        };
                                        let chan_update_opt = if let Some(forwarding_id) = forwarding_id_opt {
-                                               let chan = channel_state.by_id.get_mut(&forwarding_id).unwrap();
+                                               let chan = match channel_state.by_id.get_mut(&forwarding_id) {
+                                                       None => {
+                                                               // Channel was removed. The short_to_chan_info and by_id maps have
+                                                               // no consistency guarantees.
+                                                               break Some(("Don't have available channel for forwarding as requested.", 0x4000 | 10, None));
+                                                       },
+                                                       Some(chan) => chan
+                                               };
                                                if !chan.should_announce() && !self.default_configuration.accept_forwards_to_priv_channels {
                                                        // Note that the behavior here should be identical to the above block - we
                                                        // should NOT reveal the existence or non-existence of a private channel if
@@ -2278,10 +2344,10 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                if !chan.is_live() { // channel_disabled
                                                        break Some(("Forwarding channel is not in a ready state.", 0x1000 | 20, chan_update_opt));
                                                }
-                                               if *amt_to_forward < chan.get_counterparty_htlc_minimum_msat() { // amount_below_minimum
+                                               if *outgoing_amt_msat < chan.get_counterparty_htlc_minimum_msat() { // amount_below_minimum
                                                        break Some(("HTLC amount was below the htlc_minimum_msat", 0x1000 | 11, chan_update_opt));
                                                }
-                                               if let Err((err, code)) = chan.htlc_satisfies_config(&msg, *amt_to_forward, *outgoing_cltv_value) {
+                                               if let Err((err, code)) = chan.htlc_satisfies_config(&msg, *outgoing_amt_msat, *outgoing_cltv_value) {
                                                        break Some((err, code, chan_update_opt));
                                                }
                                                chan_update_opt
@@ -2349,7 +2415,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        /// [`MessageSendEvent::BroadcastChannelUpdate`] event.
        ///
        /// May be called with channel_state already locked!
-       fn get_channel_update_for_broadcast(&self, chan: &Channel<Signer>) -> Result<msgs::ChannelUpdate, LightningError> {
+       fn get_channel_update_for_broadcast(&self, chan: &Channel<<K::Target as KeysInterface>::Signer>) -> Result<msgs::ChannelUpdate, LightningError> {
                if !chan.should_announce() {
                        return Err(LightningError {
                                err: "Cannot broadcast a channel_update for a private channel".to_owned(),
@@ -2368,7 +2434,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        /// and thus MUST NOT be called unless the recipient of the resulting message has already
        /// provided evidence that they know about the existence of the channel.
        /// May be called with channel_state already locked!
-       fn get_channel_update_for_unicast(&self, chan: &Channel<Signer>) -> Result<msgs::ChannelUpdate, LightningError> {
+       fn get_channel_update_for_unicast(&self, chan: &Channel<<K::Target as KeysInterface>::Signer>) -> Result<msgs::ChannelUpdate, LightningError> {
                log_trace!(self.logger, "Attempting to generate channel update for channel {}", log_bytes!(chan.channel_id()));
                let short_channel_id = match chan.get_short_channel_id().or(chan.latest_inbound_scid_alias()) {
                        None => return Err(LightningError{err: "Channel not yet established".to_owned(), action: msgs::ErrorAction::IgnoreError}),
@@ -2377,7 +2443,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                self.get_channel_update_for_onion(short_channel_id, chan)
        }
-       fn get_channel_update_for_onion(&self, short_channel_id: u64, chan: &Channel<Signer>) -> Result<msgs::ChannelUpdate, LightningError> {
+       fn get_channel_update_for_onion(&self, short_channel_id: u64, chan: &Channel<<K::Target as KeysInterface>::Signer>) -> Result<msgs::ChannelUpdate, LightningError> {
                log_trace!(self.logger, "Generating channel update for channel {}", log_bytes!(chan.channel_id()));
                let were_node_one = PublicKey::from_secret_key(&self.secp_ctx, &self.our_network_key).serialize()[..] < chan.get_counterparty_node_id().serialize()[..];
 
@@ -2404,10 +2470,9 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        }
 
        // Only public for testing, this should otherwise never be called direcly
-       pub(crate) fn send_payment_along_path(&self, path: &Vec<RouteHop>, payment_params: &Option<PaymentParameters>, payment_hash: &PaymentHash, payment_secret: &Option<PaymentSecret>, total_value: u64, cur_height: u32, payment_id: PaymentId, keysend_preimage: &Option<PaymentPreimage>) -> Result<(), APIError> {
+       pub(crate) fn send_payment_along_path(&self, path: &Vec<RouteHop>, payment_params: &Option<PaymentParameters>, payment_hash: &PaymentHash, payment_secret: &Option<PaymentSecret>, total_value: u64, cur_height: u32, payment_id: PaymentId, keysend_preimage: &Option<PaymentPreimage>, session_priv_bytes: [u8; 32]) -> Result<(), APIError> {
                log_trace!(self.logger, "Attempting to send payment for path with next hop {}", path.first().unwrap().short_channel_id);
                let prng_seed = self.keys_manager.get_secure_random_bytes();
-               let session_priv_bytes = self.keys_manager.get_secure_random_bytes();
                let session_priv = SecretKey::from_slice(&session_priv_bytes[..]).expect("RNG is busted");
 
                let onion_keys = onion_utils::construct_onion_keys(&self.secp_ctx, &path, &session_priv)
@@ -2421,38 +2486,12 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                let _persistence_guard = PersistenceNotifierGuard::notify_on_drop(&self.total_consistency_lock, &self.persistence_notifier);
 
                let err: Result<(), _> = loop {
-                       let mut channel_lock = self.channel_state.lock().unwrap();
-
-                       let mut pending_outbounds = self.pending_outbound_payments.lock().unwrap();
-                       let payment_entry = pending_outbounds.entry(payment_id);
-                       if let hash_map::Entry::Occupied(payment) = &payment_entry {
-                               if !payment.get().is_retryable() {
-                                       return Err(APIError::RouteError {
-                                               err: "Payment already completed"
-                                       });
-                               }
-                       }
-
-                       let id = match channel_lock.short_to_chan_info.get(&path.first().unwrap().short_channel_id) {
+                       let id = match self.short_to_chan_info.read().unwrap().get(&path.first().unwrap().short_channel_id) {
                                None => return Err(APIError::ChannelUnavailable{err: "No channel available with first hop!".to_owned()}),
                                Some((_cp_id, chan_id)) => chan_id.clone(),
                        };
 
-                       macro_rules! insert_outbound_payment {
-                               () => {
-                                       let payment = payment_entry.or_insert_with(|| PendingOutboundPayment::Retryable {
-                                               session_privs: HashSet::new(),
-                                               pending_amt_msat: 0,
-                                               pending_fee_msat: Some(0),
-                                               payment_hash: *payment_hash,
-                                               payment_secret: *payment_secret,
-                                               starting_block_height: self.best_block.read().unwrap().height(),
-                                               total_msat: total_value,
-                                       });
-                                       assert!(payment.insert(session_priv_bytes, path));
-                               }
-                       }
-
+                       let mut channel_lock = self.channel_state.lock().unwrap();
                        let channel_state = &mut *channel_lock;
                        if let hash_map::Entry::Occupied(mut chan) = channel_state.by_id.entry(id) {
                                match {
@@ -2471,22 +2510,30 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                        payment_secret: payment_secret.clone(),
                                                        payment_params: payment_params.clone(),
                                                }, onion_packet, &self.logger),
-                                       channel_state, chan)
+                                               chan)
                                } {
                                        Some((update_add, commitment_signed, monitor_update)) => {
-                                               if let Err(e) = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
-                                                       maybe_break_monitor_err!(self, e, channel_state, chan, RAACommitmentOrder::CommitmentFirst, false, true);
-                                                       // Note that MonitorUpdateFailed here indicates (per function docs)
-                                                       // that we will resend the commitment update once monitor updating
-                                                       // is restored. Therefore, we must return an error indicating that
-                                                       // it is unsafe to retry the payment wholesale, which we do in the
-                                                       // send_payment check for MonitorUpdateFailed, below.
-                                                       insert_outbound_payment!(); // Only do this after possibly break'ing on Perm failure above.
-                                                       return Err(APIError::MonitorUpdateFailed);
+                                               let update_err = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update);
+                                               let chan_id = chan.get().channel_id();
+                                               match (update_err,
+                                                       handle_monitor_update_res!(self, update_err, chan,
+                                                               RAACommitmentOrder::CommitmentFirst, false, true))
+                                               {
+                                                       (ChannelMonitorUpdateStatus::PermanentFailure, Err(e)) => break Err(e),
+                                                       (ChannelMonitorUpdateStatus::Completed, Ok(())) => {},
+                                                       (ChannelMonitorUpdateStatus::InProgress, Err(_)) => {
+                                                               // Note that MonitorUpdateInProgress here indicates (per function
+                                                               // docs) that we will resend the commitment update once monitor
+                                                               // updating completes. Therefore, we must return an error
+                                                               // indicating that it is unsafe to retry the payment wholesale,
+                                                               // which we do in the send_payment check for
+                                                               // MonitorUpdateInProgress, below.
+                                                               return Err(APIError::MonitorUpdateInProgress);
+                                                       },
+                                                       _ => unreachable!(),
                                                }
-                                               insert_outbound_payment!();
 
-                                               log_debug!(self.logger, "Sending payment along path resulted in a commitment_signed for channel {}", log_bytes!(chan.get().channel_id()));
+                                               log_debug!(self.logger, "Sending payment along path resulted in a commitment_signed for channel {}", log_bytes!(chan_id));
                                                channel_state.pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
                                                        node_id: path.first().unwrap().pubkey,
                                                        updates: msgs::CommitmentUpdate {
@@ -2499,9 +2546,14 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                        },
                                                });
                                        },
-                                       None => { insert_outbound_payment!(); },
+                                       None => { },
                                }
-                       } else { unreachable!(); }
+                       } else {
+                               // The channel was likely removed after we fetched the id from the
+                               // `short_to_chan_info` map, but before we successfully locked the `by_id` map.
+                               // This can occur as no consistency guarantees exists between the two maps.
+                               return Err(APIError::ChannelUnavailable{err: "No channel available with first hop!".to_owned()});
+                       }
                        return Ok(());
                };
 
@@ -2518,26 +2570,32 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        /// Value parameters are provided via the last hop in route, see documentation for RouteHop
        /// fields for more info.
        ///
-       /// Note that if the payment_hash already exists elsewhere (eg you're sending a duplicative
-       /// payment), we don't do anything to stop you! We always try to ensure that if the provided
-       /// next hop knows the preimage to payment_hash they can claim an additional amount as
-       /// specified in the last hop in the route! Thus, you should probably do your own
-       /// payment_preimage tracking (which you should already be doing as they represent "proof of
-       /// payment") and prevent double-sends yourself.
+       /// If a pending payment is currently in-flight with the same [`PaymentId`] provided, this
+       /// method will error with an [`APIError::RouteError`]. Note, however, that once a payment
+       /// is no longer pending (either via [`ChannelManager::abandon_payment`], or handling of an
+       /// [`Event::PaymentSent`]) LDK will not stop you from sending a second payment with the same
+       /// [`PaymentId`].
        ///
-       /// May generate SendHTLCs message(s) event on success, which should be relayed.
+       /// Thus, in order to ensure duplicate payments are not sent, you should implement your own
+       /// tracking of payments, including state to indicate once a payment has completed. Because you
+       /// should also ensure that [`PaymentHash`]es are not re-used, for simplicity, you should
+       /// consider using the [`PaymentHash`] as the key for tracking payments. In that case, the
+       /// [`PaymentId`] should be a copy of the [`PaymentHash`] bytes.
+       ///
+       /// May generate SendHTLCs message(s) event on success, which should be relayed (e.g. via
+       /// [`PeerManager::process_events`]).
        ///
        /// Each path may have a different return value, and PaymentSendValue may return a Vec with
        /// each entry matching the corresponding-index entry in the route paths, see
        /// PaymentSendFailure for more info.
        ///
        /// In general, a path may raise:
-       ///  * APIError::RouteError when an invalid route or forwarding parameter (cltv_delta, fee,
+       ///  * [`APIError::RouteError`] when an invalid route or forwarding parameter (cltv_delta, fee,
        ///    node public key) is specified.
-       ///  * APIError::ChannelUnavailable if the next-hop channel is not available for updates
+       ///  * [`APIError::ChannelUnavailable`] if the next-hop channel is not available for updates
        ///    (including due to previous monitor update failure or new permanent monitor update
        ///    failure).
-       ///  * APIError::MonitorUpdateFailed if a new monitor update failure prevented sending the
+       ///  * [`APIError::MonitorUpdateInProgress`] if a new monitor update failure prevented sending the
        ///    relevant updates.
        ///
        /// Note that depending on the type of the PaymentSendFailure the HTLC may have been
@@ -2549,14 +2607,55 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        /// newer nodes, it will be provided to you in the invoice. If you do not have one, the Route
        /// must not contain multiple paths as multi-path payments require a recipient-provided
        /// payment_secret.
+       ///
        /// If a payment_secret *is* provided, we assume that the invoice had the payment_secret feature
        /// bit set (either as required or as available). If multiple paths are present in the Route,
        /// we assume the invoice had the basic_mpp feature set.
-       pub fn send_payment(&self, route: &Route, payment_hash: PaymentHash, payment_secret: &Option<PaymentSecret>) -> Result<PaymentId, PaymentSendFailure> {
-               self.send_payment_internal(route, payment_hash, payment_secret, None, None, None)
+       ///
+       /// [`Event::PaymentSent`]: events::Event::PaymentSent
+       /// [`PeerManager::process_events`]: crate::ln::peer_handler::PeerManager::process_events
+       pub fn send_payment(&self, route: &Route, payment_hash: PaymentHash, payment_secret: &Option<PaymentSecret>, payment_id: PaymentId) -> Result<(), PaymentSendFailure> {
+               let onion_session_privs = self.add_new_pending_payment(payment_hash, *payment_secret, payment_id, route)?;
+               self.send_payment_internal(route, payment_hash, payment_secret, None, payment_id, None, onion_session_privs)
        }
 
-       fn send_payment_internal(&self, route: &Route, payment_hash: PaymentHash, payment_secret: &Option<PaymentSecret>, keysend_preimage: Option<PaymentPreimage>, payment_id: Option<PaymentId>, recv_value_msat: Option<u64>) -> Result<PaymentId, PaymentSendFailure> {
+       #[cfg(test)]
+       pub(crate) fn test_add_new_pending_payment(&self, payment_hash: PaymentHash, payment_secret: Option<PaymentSecret>, payment_id: PaymentId, route: &Route) -> Result<Vec<[u8; 32]>, PaymentSendFailure> {
+               self.add_new_pending_payment(payment_hash, payment_secret, payment_id, route)
+       }
+
+       fn add_new_pending_payment(&self, payment_hash: PaymentHash, payment_secret: Option<PaymentSecret>, payment_id: PaymentId, route: &Route) -> Result<Vec<[u8; 32]>, PaymentSendFailure> {
+               let mut onion_session_privs = Vec::with_capacity(route.paths.len());
+               for _ in 0..route.paths.len() {
+                       onion_session_privs.push(self.keys_manager.get_secure_random_bytes());
+               }
+
+               let mut pending_outbounds = self.pending_outbound_payments.lock().unwrap();
+               match pending_outbounds.entry(payment_id) {
+                       hash_map::Entry::Occupied(_) => Err(PaymentSendFailure::ParameterError(APIError::RouteError {
+                               err: "Payment already in progress"
+                       })),
+                       hash_map::Entry::Vacant(entry) => {
+                               let payment = entry.insert(PendingOutboundPayment::Retryable {
+                                       session_privs: HashSet::new(),
+                                       pending_amt_msat: 0,
+                                       pending_fee_msat: Some(0),
+                                       payment_hash,
+                                       payment_secret,
+                                       starting_block_height: self.best_block.read().unwrap().height(),
+                                       total_msat: route.get_total_amount(),
+                               });
+
+                               for (path, session_priv_bytes) in route.paths.iter().zip(onion_session_privs.iter()) {
+                                       assert!(payment.insert(*session_priv_bytes, path));
+                               }
+
+                               Ok(onion_session_privs)
+                       },
+               }
+       }
+
+       fn send_payment_internal(&self, route: &Route, payment_hash: PaymentHash, payment_secret: &Option<PaymentSecret>, keysend_preimage: Option<PaymentPreimage>, payment_id: PaymentId, recv_value_msat: Option<u64>, onion_session_privs: Vec<[u8; 32]>) -> Result<(), PaymentSendFailure> {
                if route.paths.len() < 1 {
                        return Err(PaymentSendFailure::ParameterError(APIError::RouteError{err: "There must be at least one path to send over"}));
                }
@@ -2566,7 +2665,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                let mut total_value = 0;
                let our_node_id = self.get_our_node_id();
                let mut path_errs = Vec::with_capacity(route.paths.len());
-               let payment_id = if let Some(id) = payment_id { id } else { PaymentId(self.keys_manager.get_secure_random_bytes()) };
                'path_check: for path in route.paths.iter() {
                        if path.len() < 1 || path.len() > 20 {
                                path_errs.push(Err(APIError::RouteError{err: "Path didn't go anywhere/had bogus size"}));
@@ -2591,8 +2689,28 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                let cur_height = self.best_block.read().unwrap().height() + 1;
                let mut results = Vec::new();
-               for path in route.paths.iter() {
-                       results.push(self.send_payment_along_path(&path, &route.payment_params, &payment_hash, payment_secret, total_value, cur_height, payment_id, &keysend_preimage));
+               debug_assert_eq!(route.paths.len(), onion_session_privs.len());
+               for (path, session_priv) in route.paths.iter().zip(onion_session_privs.into_iter()) {
+                       let mut path_res = self.send_payment_along_path(&path, &route.payment_params, &payment_hash, payment_secret, total_value, cur_height, payment_id, &keysend_preimage, session_priv);
+                       match path_res {
+                               Ok(_) => {},
+                               Err(APIError::MonitorUpdateInProgress) => {
+                                       // While a MonitorUpdateInProgress is an Err(_), the payment is still
+                                       // considered "in flight" and we shouldn't remove it from the
+                                       // PendingOutboundPayment set.
+                               },
+                               Err(_) => {
+                                       let mut pending_outbounds = self.pending_outbound_payments.lock().unwrap();
+                                       if let Some(payment) = pending_outbounds.get_mut(&payment_id) {
+                                               let removed = payment.remove(&session_priv, Some(path));
+                                               debug_assert!(removed, "This can't happen as the payment has an entry for this path added by callers");
+                                       } else {
+                                               debug_assert!(false, "This can't happen as the payment was added by callers");
+                                               path_res = Err(APIError::APIMisuseError { err: "Internal error: payment disappeared during processing. Please report this bug!".to_owned() });
+                                       }
+                               }
+                       }
+                       results.push(path_res);
                }
                let mut has_ok = false;
                let mut has_err = false;
@@ -2601,8 +2719,8 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                for (res, path) in results.iter().zip(route.paths.iter()) {
                        if res.is_ok() { has_ok = true; }
                        if res.is_err() { has_err = true; }
-                       if let &Err(APIError::MonitorUpdateFailed) = res {
-                               // MonitorUpdateFailed is inherently unsafe to retry, so we call it a
+                       if let &Err(APIError::MonitorUpdateInProgress) = res {
+                               // MonitorUpdateInProgress is inherently unsafe to retry, so we call it a
                                // PartialFailure.
                                has_err = true;
                                has_ok = true;
@@ -2626,12 +2744,13 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                } else { None },
                        })
                } else if has_err {
-                       // If we failed to send any paths, we shouldn't have inserted the new PaymentId into
-                       // our `pending_outbound_payments` map at all.
-                       debug_assert!(self.pending_outbound_payments.lock().unwrap().get(&payment_id).is_none());
+                       // If we failed to send any paths, we should remove the new PaymentId from the
+                       // `pending_outbound_payments` map, as the user isn't expected to `abandon_payment`.
+                       let removed = self.pending_outbound_payments.lock().unwrap().remove(&payment_id).is_some();
+                       debug_assert!(removed, "We should always have a pending payment to remove here");
                        Err(PaymentSendFailure::AllFailedRetrySafe(results.drain(..).map(|r| r.unwrap_err()).collect()))
                } else {
-                       Ok(payment_id)
+                       Ok(())
                }
        }
 
@@ -2655,44 +2774,55 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        }
                }
 
+               let mut onion_session_privs = Vec::with_capacity(route.paths.len());
+               for _ in 0..route.paths.len() {
+                       onion_session_privs.push(self.keys_manager.get_secure_random_bytes());
+               }
+
                let (total_msat, payment_hash, payment_secret) = {
-                       let outbounds = self.pending_outbound_payments.lock().unwrap();
-                       if let Some(payment) = outbounds.get(&payment_id) {
-                               match payment {
-                                       PendingOutboundPayment::Retryable {
-                                               total_msat, payment_hash, payment_secret, pending_amt_msat, ..
-                                       } => {
-                                               let retry_amt_msat: u64 = route.paths.iter().map(|path| path.last().unwrap().fee_msat).sum();
-                                               if retry_amt_msat + *pending_amt_msat > *total_msat * (100 + RETRY_OVERFLOW_PERCENTAGE) / 100 {
+                       let mut outbounds = self.pending_outbound_payments.lock().unwrap();
+                       match outbounds.get_mut(&payment_id) {
+                               Some(payment) => {
+                                       let res = match payment {
+                                               PendingOutboundPayment::Retryable {
+                                                       total_msat, payment_hash, payment_secret, pending_amt_msat, ..
+                                               } => {
+                                                       let retry_amt_msat: u64 = route.paths.iter().map(|path| path.last().unwrap().fee_msat).sum();
+                                                       if retry_amt_msat + *pending_amt_msat > *total_msat * (100 + RETRY_OVERFLOW_PERCENTAGE) / 100 {
+                                                               return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
+                                                                       err: format!("retry_amt_msat of {} will put pending_amt_msat (currently: {}) more than 10% over total_payment_amt_msat of {}", retry_amt_msat, pending_amt_msat, total_msat).to_string()
+                                                               }))
+                                                       }
+                                                       (*total_msat, *payment_hash, *payment_secret)
+                                               },
+                                               PendingOutboundPayment::Legacy { .. } => {
                                                        return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
-                                                               err: format!("retry_amt_msat of {} will put pending_amt_msat (currently: {}) more than 10% over total_payment_amt_msat of {}", retry_amt_msat, pending_amt_msat, total_msat).to_string()
+                                                               err: "Unable to retry payments that were initially sent on LDK versions prior to 0.0.102".to_string()
                                                        }))
-                                               }
-                                               (*total_msat, *payment_hash, *payment_secret)
-                                       },
-                                       PendingOutboundPayment::Legacy { .. } => {
-                                               return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
-                                                       err: "Unable to retry payments that were initially sent on LDK versions prior to 0.0.102".to_string()
-                                               }))
-                                       },
-                                       PendingOutboundPayment::Fulfilled { .. } => {
-                                               return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
-                                                       err: "Payment already completed".to_owned()
-                                               }));
-                                       },
-                                       PendingOutboundPayment::Abandoned { .. } => {
-                                               return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
-                                                       err: "Payment already abandoned (with some HTLCs still pending)".to_owned()
-                                               }));
-                                       },
-                               }
-                       } else {
-                               return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
-                                       err: format!("Payment with ID {} not found", log_bytes!(payment_id.0)),
-                               }))
+                                               },
+                                               PendingOutboundPayment::Fulfilled { .. } => {
+                                                       return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
+                                                               err: "Payment already completed".to_owned()
+                                                       }));
+                                               },
+                                               PendingOutboundPayment::Abandoned { .. } => {
+                                                       return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
+                                                               err: "Payment already abandoned (with some HTLCs still pending)".to_owned()
+                                                       }));
+                                               },
+                                       };
+                                       for (path, session_priv_bytes) in route.paths.iter().zip(onion_session_privs.iter()) {
+                                               assert!(payment.insert(*session_priv_bytes, path));
+                                       }
+                                       res
+                               },
+                               None =>
+                                       return Err(PaymentSendFailure::ParameterError(APIError::APIMisuseError {
+                                               err: format!("Payment with ID {} not found", log_bytes!(payment_id.0)),
+                                       })),
                        }
                };
-               return self.send_payment_internal(route, payment_hash, &payment_secret, None, Some(payment_id), Some(total_msat)).map(|_| ())
+               self.send_payment_internal(route, payment_hash, &payment_secret, None, payment_id, Some(total_msat), onion_session_privs)
        }
 
        /// Signals that no further retries for the given payment will occur.
@@ -2732,7 +2862,8 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        /// would be able to guess -- otherwise, an intermediate node may claim the payment and it will
        /// never reach the recipient.
        ///
-       /// See [`send_payment`] documentation for more details on the return value of this function.
+       /// See [`send_payment`] documentation for more details on the return value of this function
+       /// and idempotency guarantees provided by the [`PaymentId`] key.
        ///
        /// Similar to regular payments, you MUST NOT reuse a `payment_preimage` value. See
        /// [`send_payment`] for more information about the risks of duplicate preimage usage.
@@ -2740,14 +2871,16 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        /// Note that `route` must have exactly one path.
        ///
        /// [`send_payment`]: Self::send_payment
-       pub fn send_spontaneous_payment(&self, route: &Route, payment_preimage: Option<PaymentPreimage>) -> Result<(PaymentHash, PaymentId), PaymentSendFailure> {
+       pub fn send_spontaneous_payment(&self, route: &Route, payment_preimage: Option<PaymentPreimage>, payment_id: PaymentId) -> Result<PaymentHash, PaymentSendFailure> {
                let preimage = match payment_preimage {
                        Some(p) => p,
                        None => PaymentPreimage(self.keys_manager.get_secure_random_bytes()),
                };
                let payment_hash = PaymentHash(Sha256::hash(&preimage.0).into_inner());
-               match self.send_payment_internal(route, payment_hash, &None, Some(preimage), None, None) {
-                       Ok(payment_id) => Ok((payment_hash, payment_id)),
+               let onion_session_privs = self.add_new_pending_payment(payment_hash, None, payment_id, &route)?;
+
+               match self.send_payment_internal(route, payment_hash, &None, Some(preimage), payment_id, None, onion_session_privs) {
+                       Ok(()) => Ok(payment_hash),
                        Err(e) => Err(e)
                }
        }
@@ -2767,9 +2900,10 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                }
 
                let route = Route { paths: vec![hops], payment_params: None };
+               let onion_session_privs = self.add_new_pending_payment(payment_hash, None, payment_id, &route)?;
 
-               match self.send_payment_internal(&route, payment_hash, &None, None, Some(payment_id), None) {
-                       Ok(payment_id) => Ok((payment_hash, payment_id)),
+               match self.send_payment_internal(&route, payment_hash, &None, None, payment_id, None, onion_session_privs) {
+                       Ok(()) => Ok((payment_hash, payment_id)),
                        Err(e) => Err(e)
                }
        }
@@ -2791,7 +2925,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
        /// Handles the generation of a funding transaction, optionally (for tests) with a function
        /// which checks the correctness of the funding transaction given the associated channel.
-       fn funding_transaction_generated_intern<FundingOutput: Fn(&Channel<Signer>, &Transaction) -> Result<OutPoint, APIError>>(
+       fn funding_transaction_generated_intern<FundingOutput: Fn(&Channel<<K::Target as KeysInterface>::Signer>, &Transaction) -> Result<OutPoint, APIError>>(
                &self, temporary_channel_id: &[u8; 32], _counterparty_node_id: &PublicKey, funding_transaction: Transaction, find_funding_output: FundingOutput
        ) -> Result<(), APIError> {
                let (chan, msg) = {
@@ -2889,7 +3023,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        // Transactions are evaluated as final by network mempools at the next block. However, the modules
                        // constituting our Lightning node might not have perfect sync about their blockchain views. Thus, if
                        // the wallet module is in advance on the LDK view, allow one more block of headroom.
-                       // TODO: updated if/when https://github.com/rust-bitcoin/rust-bitcoin/pull/994 landed and rust-bitcoin bumped.
                        if !funding_transaction.input.iter().all(|input| input.sequence == Sequence::MAX) && LockTime::from(funding_transaction.lock_time).is_block_height() && funding_transaction.lock_time.0 > height + 2 {
                                return Err(APIError::APIMisuseError {
                                        err: "Funding transaction absolute timelock is non-final".to_owned()
@@ -3002,92 +3135,97 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                let mut phantom_receives: Vec<(u64, OutPoint, Vec<(PendingHTLCInfo, u64)>)> = Vec::new();
                let mut handle_errors = Vec::new();
                {
-                       let mut channel_state_lock = self.channel_state.lock().unwrap();
-                       let channel_state = &mut *channel_state_lock;
+                       let mut forward_htlcs = HashMap::new();
+                       mem::swap(&mut forward_htlcs, &mut self.forward_htlcs.lock().unwrap());
 
-                       for (short_chan_id, mut pending_forwards) in channel_state.forward_htlcs.drain() {
+                       for (short_chan_id, mut pending_forwards) in forward_htlcs {
+                               let mut channel_state_lock = self.channel_state.lock().unwrap();
+                               let channel_state = &mut *channel_state_lock;
                                if short_chan_id != 0 {
-                                       let forward_chan_id = match channel_state.short_to_chan_info.get(&short_chan_id) {
-                                               Some((_cp_id, chan_id)) => chan_id.clone(),
-                                               None => {
+                                       macro_rules! forwarding_channel_not_found {
+                                               () => {
                                                        for forward_info in pending_forwards.drain(..) {
                                                                match forward_info {
-                                                                       HTLCForwardInfo::AddHTLC { prev_short_channel_id, prev_htlc_id, forward_info: PendingHTLCInfo {
-                                                                               routing, incoming_shared_secret, payment_hash, amt_to_forward, outgoing_cltv_value },
-                                                                               prev_funding_outpoint } => {
-                                                                                       macro_rules! failure_handler {
-                                                                                               ($msg: expr, $err_code: expr, $err_data: expr, $phantom_ss: expr, $next_hop_unknown: expr) => {
-                                                                                                       log_info!(self.logger, "Failed to accept/forward incoming HTLC: {}", $msg);
-
-                                                                                                       let htlc_source = HTLCSource::PreviousHopData(HTLCPreviousHopData {
-                                                                                                               short_channel_id: prev_short_channel_id,
-                                                                                                               outpoint: prev_funding_outpoint,
-                                                                                                               htlc_id: prev_htlc_id,
-                                                                                                               incoming_packet_shared_secret: incoming_shared_secret,
-                                                                                                               phantom_shared_secret: $phantom_ss,
-                                                                                                       });
-
-                                                                                                       let reason = if $next_hop_unknown {
-                                                                                                               HTLCDestination::UnknownNextHop { requested_forward_scid: short_chan_id }
-                                                                                                       } else {
-                                                                                                               HTLCDestination::FailedPayment{ payment_hash }
-                                                                                                       };
-
-                                                                                                       failed_forwards.push((htlc_source, payment_hash,
-                                                                                                               HTLCFailReason::Reason { failure_code: $err_code, data: $err_data },
-                                                                                                               reason
-                                                                                                       ));
-                                                                                                       continue;
-                                                                                               }
+                                                                       HTLCForwardInfo::AddHTLC(PendingAddHTLCInfo {
+                                                                               prev_short_channel_id, prev_htlc_id, prev_funding_outpoint,
+                                                                               forward_info: PendingHTLCInfo {
+                                                                                       routing, incoming_shared_secret, payment_hash, outgoing_amt_msat,
+                                                                                       outgoing_cltv_value, incoming_amt_msat: _
+                                                                               }
+                                                                       }) => {
+                                                                               macro_rules! failure_handler {
+                                                                                       ($msg: expr, $err_code: expr, $err_data: expr, $phantom_ss: expr, $next_hop_unknown: expr) => {
+                                                                                               log_info!(self.logger, "Failed to accept/forward incoming HTLC: {}", $msg);
+
+                                                                                               let htlc_source = HTLCSource::PreviousHopData(HTLCPreviousHopData {
+                                                                                                       short_channel_id: prev_short_channel_id,
+                                                                                                       outpoint: prev_funding_outpoint,
+                                                                                                       htlc_id: prev_htlc_id,
+                                                                                                       incoming_packet_shared_secret: incoming_shared_secret,
+                                                                                                       phantom_shared_secret: $phantom_ss,
+                                                                                               });
+
+                                                                                               let reason = if $next_hop_unknown {
+                                                                                                       HTLCDestination::UnknownNextHop { requested_forward_scid: short_chan_id }
+                                                                                               } else {
+                                                                                                       HTLCDestination::FailedPayment{ payment_hash }
+                                                                                               };
+
+                                                                                               failed_forwards.push((htlc_source, payment_hash,
+                                                                                                       HTLCFailReason::Reason { failure_code: $err_code, data: $err_data },
+                                                                                                       reason
+                                                                                               ));
+                                                                                               continue;
                                                                                        }
-                                                                                       macro_rules! fail_forward {
-                                                                                               ($msg: expr, $err_code: expr, $err_data: expr, $phantom_ss: expr) => {
-                                                                                                       {
-                                                                                                               failure_handler!($msg, $err_code, $err_data, $phantom_ss, true);
-                                                                                                       }
+                                                                               }
+                                                                               macro_rules! fail_forward {
+                                                                                       ($msg: expr, $err_code: expr, $err_data: expr, $phantom_ss: expr) => {
+                                                                                               {
+                                                                                                       failure_handler!($msg, $err_code, $err_data, $phantom_ss, true);
                                                                                                }
                                                                                        }
-                                                                                       macro_rules! failed_payment {
-                                                                                               ($msg: expr, $err_code: expr, $err_data: expr, $phantom_ss: expr) => {
-                                                                                                       {
-                                                                                                               failure_handler!($msg, $err_code, $err_data, $phantom_ss, false);
-                                                                                                       }
+                                                                               }
+                                                                               macro_rules! failed_payment {
+                                                                                       ($msg: expr, $err_code: expr, $err_data: expr, $phantom_ss: expr) => {
+                                                                                               {
+                                                                                                       failure_handler!($msg, $err_code, $err_data, $phantom_ss, false);
                                                                                                }
                                                                                        }
-                                                                                       if let PendingHTLCRouting::Forward { onion_packet, .. } = routing {
-                                                                                               let phantom_secret_res = self.keys_manager.get_node_secret(Recipient::PhantomNode);
-                                                                                               if phantom_secret_res.is_ok() && fake_scid::is_valid_phantom(&self.fake_scid_rand_bytes, short_chan_id) {
-                                                                                                       let phantom_shared_secret = SharedSecret::new(&onion_packet.public_key.unwrap(), &phantom_secret_res.unwrap()).secret_bytes();
-                                                                                                       let next_hop = match onion_utils::decode_next_payment_hop(phantom_shared_secret, &onion_packet.hop_data, onion_packet.hmac, payment_hash) {
-                                                                                                               Ok(res) => res,
-                                                                                                               Err(onion_utils::OnionDecodeErr::Malformed { err_msg, err_code }) => {
-                                                                                                                       let sha256_of_onion = Sha256::hash(&onion_packet.hop_data).into_inner();
-                                                                                                                       // In this scenario, the phantom would have sent us an
-                                                                                                                       // `update_fail_malformed_htlc`, meaning here we encrypt the error as
-                                                                                                                       // if it came from us (the second-to-last hop) but contains the sha256
-                                                                                                                       // of the onion.
-                                                                                                                       failed_payment!(err_msg, err_code, sha256_of_onion.to_vec(), None);
-                                                                                                               },
-                                                                                                               Err(onion_utils::OnionDecodeErr::Relay { err_msg, err_code }) => {
-                                                                                                                       failed_payment!(err_msg, err_code, Vec::new(), Some(phantom_shared_secret));
-                                                                                                               },
-                                                                                                       };
-                                                                                                       match next_hop {
-                                                                                                               onion_utils::Hop::Receive(hop_data) => {
-                                                                                                                       match self.construct_recv_pending_htlc_info(hop_data, incoming_shared_secret, payment_hash, amt_to_forward, outgoing_cltv_value, Some(phantom_shared_secret)) {
-                                                                                                                               Ok(info) => phantom_receives.push((prev_short_channel_id, prev_funding_outpoint, vec![(info, prev_htlc_id)])),
-                                                                                                                               Err(ReceiveError { err_code, err_data, msg }) => failed_payment!(msg, err_code, err_data, Some(phantom_shared_secret))
-                                                                                                                       }
-                                                                                                               },
-                                                                                                               _ => panic!(),
-                                                                                                       }
-                                                                                               } else {
-                                                                                                       fail_forward!(format!("Unknown short channel id {} for forward HTLC", short_chan_id), 0x4000 | 10, Vec::new(), None);
+                                                                               }
+                                                                               if let PendingHTLCRouting::Forward { onion_packet, .. } = routing {
+                                                                                       let phantom_secret_res = self.keys_manager.get_node_secret(Recipient::PhantomNode);
+                                                                                       if phantom_secret_res.is_ok() && fake_scid::is_valid_phantom(&self.fake_scid_rand_bytes, short_chan_id, &self.genesis_hash) {
+                                                                                               let phantom_shared_secret = SharedSecret::new(&onion_packet.public_key.unwrap(), &phantom_secret_res.unwrap()).secret_bytes();
+                                                                                               let next_hop = match onion_utils::decode_next_payment_hop(phantom_shared_secret, &onion_packet.hop_data, onion_packet.hmac, payment_hash) {
+                                                                                                       Ok(res) => res,
+                                                                                                       Err(onion_utils::OnionDecodeErr::Malformed { err_msg, err_code }) => {
+                                                                                                               let sha256_of_onion = Sha256::hash(&onion_packet.hop_data).into_inner();
+                                                                                                               // In this scenario, the phantom would have sent us an
+                                                                                                               // `update_fail_malformed_htlc`, meaning here we encrypt the error as
+                                                                                                               // if it came from us (the second-to-last hop) but contains the sha256
+                                                                                                               // of the onion.
+                                                                                                               failed_payment!(err_msg, err_code, sha256_of_onion.to_vec(), None);
+                                                                                                       },
+                                                                                                       Err(onion_utils::OnionDecodeErr::Relay { err_msg, err_code }) => {
+                                                                                                               failed_payment!(err_msg, err_code, Vec::new(), Some(phantom_shared_secret));
+                                                                                                       },
+                                                                                               };
+                                                                                               match next_hop {
+                                                                                                       onion_utils::Hop::Receive(hop_data) => {
+                                                                                                               match self.construct_recv_pending_htlc_info(hop_data, incoming_shared_secret, payment_hash, outgoing_amt_msat, outgoing_cltv_value, Some(phantom_shared_secret)) {
+                                                                                                                       Ok(info) => phantom_receives.push((prev_short_channel_id, prev_funding_outpoint, vec![(info, prev_htlc_id)])),
+                                                                                                                       Err(ReceiveError { err_code, err_data, msg }) => failed_payment!(msg, err_code, err_data, Some(phantom_shared_secret))
+                                                                                                               }
+                                                                                                       },
+                                                                                                       _ => panic!(),
                                                                                                }
                                                                                        } else {
                                                                                                fail_forward!(format!("Unknown short channel id {} for forward HTLC", short_chan_id), 0x4000 | 10, Vec::new(), None);
                                                                                        }
-                                                                               },
+                                                                               } else {
+                                                                                       fail_forward!(format!("Unknown short channel id {} for forward HTLC", short_chan_id), 0x4000 | 10, Vec::new(), None);
+                                                                               }
+                                                                       },
                                                                        HTLCForwardInfo::FailHTLC { .. } => {
                                                                                // Channel went away before we could fail it. This implies
                                                                                // the channel is now on chain and our counterparty is
@@ -3096,140 +3234,158 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                                        }
                                                                }
                                                        }
+                                               }
+                                       }
+                                       let forward_chan_id = match self.short_to_chan_info.read().unwrap().get(&short_chan_id) {
+                                               Some((_cp_id, chan_id)) => chan_id.clone(),
+                                               None => {
+                                                       forwarding_channel_not_found!();
                                                        continue;
                                                }
                                        };
-                                       if let hash_map::Entry::Occupied(mut chan) = channel_state.by_id.entry(forward_chan_id) {
-                                               let mut add_htlc_msgs = Vec::new();
-                                               let mut fail_htlc_msgs = Vec::new();
-                                               for forward_info in pending_forwards.drain(..) {
-                                                       match forward_info {
-                                                               HTLCForwardInfo::AddHTLC { prev_short_channel_id, prev_htlc_id, forward_info: PendingHTLCInfo {
-                                                                               routing: PendingHTLCRouting::Forward {
-                                                                                       onion_packet, ..
-                                                                               }, incoming_shared_secret, payment_hash, amt_to_forward, outgoing_cltv_value },
-                                                                               prev_funding_outpoint } => {
-                                                                       log_trace!(self.logger, "Adding HTLC from short id {} with payment_hash {} to channel with short id {} after delay", prev_short_channel_id, log_bytes!(payment_hash.0), short_chan_id);
-                                                                       let htlc_source = HTLCSource::PreviousHopData(HTLCPreviousHopData {
-                                                                               short_channel_id: prev_short_channel_id,
-                                                                               outpoint: prev_funding_outpoint,
-                                                                               htlc_id: prev_htlc_id,
-                                                                               incoming_packet_shared_secret: incoming_shared_secret,
-                                                                               // Phantom payments are only PendingHTLCRouting::Receive.
-                                                                               phantom_shared_secret: None,
-                                                                       });
-                                                                       match chan.get_mut().send_htlc(amt_to_forward, payment_hash, outgoing_cltv_value, htlc_source.clone(), onion_packet, &self.logger) {
-                                                                               Err(e) => {
-                                                                                       if let ChannelError::Ignore(msg) = e {
-                                                                                               log_trace!(self.logger, "Failed to forward HTLC with payment_hash {}: {}", log_bytes!(payment_hash.0), msg);
-                                                                                       } else {
-                                                                                               panic!("Stated return value requirements in send_htlc() were not met");
-                                                                                       }
-                                                                                       let (failure_code, data) = self.get_htlc_temp_fail_err_and_data(0x1000|7, short_chan_id, chan.get());
-                                                                                       failed_forwards.push((htlc_source, payment_hash,
-                                                                                               HTLCFailReason::Reason { failure_code, data },
-                                                                                               HTLCDestination::NextHopChannel { node_id: Some(chan.get().get_counterparty_node_id()), channel_id: forward_chan_id }
-                                                                                       ));
-                                                                                       continue;
+                                       match channel_state.by_id.entry(forward_chan_id) {
+                                               hash_map::Entry::Vacant(_) => {
+                                                       forwarding_channel_not_found!();
+                                                       continue;
+                                               },
+                                               hash_map::Entry::Occupied(mut chan) => {
+                                                       let mut add_htlc_msgs = Vec::new();
+                                                       let mut fail_htlc_msgs = Vec::new();
+                                                       for forward_info in pending_forwards.drain(..) {
+                                                               match forward_info {
+                                                                       HTLCForwardInfo::AddHTLC(PendingAddHTLCInfo {
+                                                                               prev_short_channel_id, prev_htlc_id, prev_funding_outpoint ,
+                                                                               forward_info: PendingHTLCInfo {
+                                                                                       incoming_shared_secret, payment_hash, outgoing_amt_msat, outgoing_cltv_value,
+                                                                                       routing: PendingHTLCRouting::Forward { onion_packet, .. }, incoming_amt_msat: _,
                                                                                },
-                                                                               Ok(update_add) => {
-                                                                                       match update_add {
-                                                                                               Some(msg) => { add_htlc_msgs.push(msg); },
-                                                                                               None => {
-                                                                                                       // Nothing to do here...we're waiting on a remote
-                                                                                                       // revoke_and_ack before we can add anymore HTLCs. The Channel
-                                                                                                       // will automatically handle building the update_add_htlc and
-                                                                                                       // commitment_signed messages when we can.
-                                                                                                       // TODO: Do some kind of timer to set the channel as !is_live()
-                                                                                                       // as we don't really want others relying on us relaying through
-                                                                                                       // this channel currently :/.
+                                                                       }) => {
+                                                                               log_trace!(self.logger, "Adding HTLC from short id {} with payment_hash {} to channel with short id {} after delay", prev_short_channel_id, log_bytes!(payment_hash.0), short_chan_id);
+                                                                               let htlc_source = HTLCSource::PreviousHopData(HTLCPreviousHopData {
+                                                                                       short_channel_id: prev_short_channel_id,
+                                                                                       outpoint: prev_funding_outpoint,
+                                                                                       htlc_id: prev_htlc_id,
+                                                                                       incoming_packet_shared_secret: incoming_shared_secret,
+                                                                                       // Phantom payments are only PendingHTLCRouting::Receive.
+                                                                                       phantom_shared_secret: None,
+                                                                               });
+                                                                               match chan.get_mut().send_htlc(outgoing_amt_msat, payment_hash, outgoing_cltv_value, htlc_source.clone(), onion_packet, &self.logger) {
+                                                                                       Err(e) => {
+                                                                                               if let ChannelError::Ignore(msg) = e {
+                                                                                                       log_trace!(self.logger, "Failed to forward HTLC with payment_hash {}: {}", log_bytes!(payment_hash.0), msg);
+                                                                                               } else {
+                                                                                                       panic!("Stated return value requirements in send_htlc() were not met");
+                                                                                               }
+                                                                                               let (failure_code, data) = self.get_htlc_temp_fail_err_and_data(0x1000|7, short_chan_id, chan.get());
+                                                                                               failed_forwards.push((htlc_source, payment_hash,
+                                                                                                       HTLCFailReason::Reason { failure_code, data },
+                                                                                                       HTLCDestination::NextHopChannel { node_id: Some(chan.get().get_counterparty_node_id()), channel_id: forward_chan_id }
+                                                                                               ));
+                                                                                               continue;
+                                                                                       },
+                                                                                       Ok(update_add) => {
+                                                                                               match update_add {
+                                                                                                       Some(msg) => { add_htlc_msgs.push(msg); },
+                                                                                                       None => {
+                                                                                                               // Nothing to do here...we're waiting on a remote
+                                                                                                               // revoke_and_ack before we can add anymore HTLCs. The Channel
+                                                                                                               // will automatically handle building the update_add_htlc and
+                                                                                                               // commitment_signed messages when we can.
+                                                                                                               // TODO: Do some kind of timer to set the channel as !is_live()
+                                                                                                               // as we don't really want others relying on us relaying through
+                                                                                                               // this channel currently :/.
+                                                                                                       }
                                                                                                }
                                                                                        }
                                                                                }
-                                                                       }
-                                                               },
-                                                               HTLCForwardInfo::AddHTLC { .. } => {
-                                                                       panic!("short_channel_id != 0 should imply any pending_forward entries are of type Forward");
-                                                               },
-                                                               HTLCForwardInfo::FailHTLC { htlc_id, err_packet } => {
-                                                                       log_trace!(self.logger, "Failing HTLC back to channel with short id {} (backward HTLC ID {}) after delay", short_chan_id, htlc_id);
-                                                                       match chan.get_mut().get_update_fail_htlc(htlc_id, err_packet, &self.logger) {
-                                                                               Err(e) => {
-                                                                                       if let ChannelError::Ignore(msg) = e {
-                                                                                               log_trace!(self.logger, "Failed to fail HTLC with ID {} backwards to short_id {}: {}", htlc_id, short_chan_id, msg);
-                                                                                       } else {
-                                                                                               panic!("Stated return value requirements in get_update_fail_htlc() were not met");
+                                                                       },
+                                                                       HTLCForwardInfo::AddHTLC { .. } => {
+                                                                               panic!("short_channel_id != 0 should imply any pending_forward entries are of type Forward");
+                                                                       },
+                                                                       HTLCForwardInfo::FailHTLC { htlc_id, err_packet } => {
+                                                                               log_trace!(self.logger, "Failing HTLC back to channel with short id {} (backward HTLC ID {}) after delay", short_chan_id, htlc_id);
+                                                                               match chan.get_mut().get_update_fail_htlc(htlc_id, err_packet, &self.logger) {
+                                                                                       Err(e) => {
+                                                                                               if let ChannelError::Ignore(msg) = e {
+                                                                                                       log_trace!(self.logger, "Failed to fail HTLC with ID {} backwards to short_id {}: {}", htlc_id, short_chan_id, msg);
+                                                                                               } else {
+                                                                                                       panic!("Stated return value requirements in get_update_fail_htlc() were not met");
+                                                                                               }
+                                                                                               // fail-backs are best-effort, we probably already have one
+                                                                                               // pending, and if not that's OK, if not, the channel is on
+                                                                                               // the chain and sending the HTLC-Timeout is their problem.
+                                                                                               continue;
+                                                                                       },
+                                                                                       Ok(Some(msg)) => { fail_htlc_msgs.push(msg); },
+                                                                                       Ok(None) => {
+                                                                                               // Nothing to do here...we're waiting on a remote
+                                                                                               // revoke_and_ack before we can update the commitment
+                                                                                               // transaction. The Channel will automatically handle
+                                                                                               // building the update_fail_htlc and commitment_signed
+                                                                                               // messages when we can.
+                                                                                               // We don't need any kind of timer here as they should fail
+                                                                                               // the channel onto the chain if they can't get our
+                                                                                               // update_fail_htlc in time, it's not our problem.
                                                                                        }
-                                                                                       // fail-backs are best-effort, we probably already have one
-                                                                                       // pending, and if not that's OK, if not, the channel is on
-                                                                                       // the chain and sending the HTLC-Timeout is their problem.
-                                                                                       continue;
-                                                                               },
-                                                                               Ok(Some(msg)) => { fail_htlc_msgs.push(msg); },
-                                                                               Ok(None) => {
-                                                                                       // Nothing to do here...we're waiting on a remote
-                                                                                       // revoke_and_ack before we can update the commitment
-                                                                                       // transaction. The Channel will automatically handle
-                                                                                       // building the update_fail_htlc and commitment_signed
-                                                                                       // messages when we can.
-                                                                                       // We don't need any kind of timer here as they should fail
-                                                                                       // the channel onto the chain if they can't get our
-                                                                                       // update_fail_htlc in time, it's not our problem.
                                                                                }
-                                                                       }
-                                                               },
+                                                                       },
+                                                               }
                                                        }
-                                               }
 
-                                               if !add_htlc_msgs.is_empty() || !fail_htlc_msgs.is_empty() {
-                                                       let (commitment_msg, monitor_update) = match chan.get_mut().send_commitment(&self.logger) {
-                                                               Ok(res) => res,
-                                                               Err(e) => {
-                                                                       // We surely failed send_commitment due to bad keys, in that case
-                                                                       // close channel and then send error message to peer.
-                                                                       let counterparty_node_id = chan.get().get_counterparty_node_id();
-                                                                       let err: Result<(), _>  = match e {
-                                                                               ChannelError::Ignore(_) | ChannelError::Warn(_) => {
-                                                                                       panic!("Stated return value requirements in send_commitment() were not met");
-                                                                               }
-                                                                               ChannelError::Close(msg) => {
-                                                                                       log_trace!(self.logger, "Closing channel {} due to Close-required error: {}", log_bytes!(chan.key()[..]), msg);
-                                                                                       let mut channel = remove_channel!(self, channel_state, chan);
-                                                                                       // ChannelClosed event is generated by handle_error for us.
-                                                                                       Err(MsgHandleErrInternal::from_finish_shutdown(msg, channel.channel_id(), channel.get_user_id(), channel.force_shutdown(true), self.get_channel_update_for_broadcast(&channel).ok()))
-                                                                               },
-                                                                       };
-                                                                       handle_errors.push((counterparty_node_id, err));
-                                                                       continue;
+                                                       if !add_htlc_msgs.is_empty() || !fail_htlc_msgs.is_empty() {
+                                                               let (commitment_msg, monitor_update) = match chan.get_mut().send_commitment(&self.logger) {
+                                                                       Ok(res) => res,
+                                                                       Err(e) => {
+                                                                               // We surely failed send_commitment due to bad keys, in that case
+                                                                               // close channel and then send error message to peer.
+                                                                               let counterparty_node_id = chan.get().get_counterparty_node_id();
+                                                                               let err: Result<(), _>  = match e {
+                                                                                       ChannelError::Ignore(_) | ChannelError::Warn(_) => {
+                                                                                               panic!("Stated return value requirements in send_commitment() were not met");
+                                                                                       }
+                                                                                       ChannelError::Close(msg) => {
+                                                                                               log_trace!(self.logger, "Closing channel {} due to Close-required error: {}", log_bytes!(chan.key()[..]), msg);
+                                                                                               let mut channel = remove_channel!(self, chan);
+                                                                                               // ChannelClosed event is generated by handle_error for us.
+                                                                                               Err(MsgHandleErrInternal::from_finish_shutdown(msg, channel.channel_id(), channel.get_user_id(), channel.force_shutdown(true), self.get_channel_update_for_broadcast(&channel).ok()))
+                                                                                       },
+                                                                               };
+                                                                               handle_errors.push((counterparty_node_id, err));
+                                                                               continue;
+                                                                       }
+                                                               };
+                                                               match self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
+                                                                       ChannelMonitorUpdateStatus::Completed => {},
+                                                                       e => {
+                                                                               handle_errors.push((chan.get().get_counterparty_node_id(), handle_monitor_update_res!(self, e, chan, RAACommitmentOrder::CommitmentFirst, false, true)));
+                                                                               continue;
+                                                                       }
                                                                }
-                                                       };
-                                                       if let Err(e) = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
-                                                               handle_errors.push((chan.get().get_counterparty_node_id(), handle_monitor_err!(self, e, channel_state, chan, RAACommitmentOrder::CommitmentFirst, false, true)));
-                                                               continue;
+                                                               log_debug!(self.logger, "Forwarding HTLCs resulted in a commitment update with {} HTLCs added and {} HTLCs failed for channel {}",
+                                                                       add_htlc_msgs.len(), fail_htlc_msgs.len(), log_bytes!(chan.get().channel_id()));
+                                                               channel_state.pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
+                                                                       node_id: chan.get().get_counterparty_node_id(),
+                                                                       updates: msgs::CommitmentUpdate {
+                                                                               update_add_htlcs: add_htlc_msgs,
+                                                                               update_fulfill_htlcs: Vec::new(),
+                                                                               update_fail_htlcs: fail_htlc_msgs,
+                                                                               update_fail_malformed_htlcs: Vec::new(),
+                                                                               update_fee: None,
+                                                                               commitment_signed: commitment_msg,
+                                                                       },
+                                                               });
                                                        }
-                                                       log_debug!(self.logger, "Forwarding HTLCs resulted in a commitment update with {} HTLCs added and {} HTLCs failed for channel {}",
-                                                               add_htlc_msgs.len(), fail_htlc_msgs.len(), log_bytes!(chan.get().channel_id()));
-                                                       channel_state.pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
-                                                               node_id: chan.get().get_counterparty_node_id(),
-                                                               updates: msgs::CommitmentUpdate {
-                                                                       update_add_htlcs: add_htlc_msgs,
-                                                                       update_fulfill_htlcs: Vec::new(),
-                                                                       update_fail_htlcs: fail_htlc_msgs,
-                                                                       update_fail_malformed_htlcs: Vec::new(),
-                                                                       update_fee: None,
-                                                                       commitment_signed: commitment_msg,
-                                                               },
-                                                       });
                                                }
-                                       } else {
-                                               unreachable!();
                                        }
                                } else {
                                        for forward_info in pending_forwards.drain(..) {
                                                match forward_info {
-                                                       HTLCForwardInfo::AddHTLC { prev_short_channel_id, prev_htlc_id, forward_info: PendingHTLCInfo {
-                                                                       routing, incoming_shared_secret, payment_hash, amt_to_forward, .. },
-                                                                       prev_funding_outpoint } => {
+                                                       HTLCForwardInfo::AddHTLC(PendingAddHTLCInfo {
+                                                               prev_short_channel_id, prev_htlc_id, prev_funding_outpoint,
+                                                               forward_info: PendingHTLCInfo {
+                                                                       routing, incoming_shared_secret, payment_hash, outgoing_amt_msat, ..
+                                                               }
+                                                       }) => {
                                                                let (cltv_expiry, onion_payload, payment_data, phantom_shared_secret) = match routing {
                                                                        PendingHTLCRouting::Receive { payment_data, incoming_cltv_expiry, phantom_shared_secret } => {
                                                                                let _legacy_hop_data = Some(payment_data.clone());
@@ -3249,9 +3405,9 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                                                incoming_packet_shared_secret: incoming_shared_secret,
                                                                                phantom_shared_secret,
                                                                        },
-                                                                       value: amt_to_forward,
+                                                                       value: outgoing_amt_msat,
                                                                        timer_ticks: 0,
-                                                                       total_msat: if let Some(data) = &payment_data { data.total_msat } else { amt_to_forward },
+                                                                       total_msat: if let Some(data) = &payment_data { data.total_msat } else { outgoing_amt_msat },
                                                                        cltv_expiry,
                                                                        onion_payload,
                                                                };
@@ -3358,7 +3514,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                                                                                e.insert((purpose.clone(), vec![claimable_htlc]));
                                                                                                                new_events.push(events::Event::PaymentReceived {
                                                                                                                        payment_hash,
-                                                                                                                       amount_msat: amt_to_forward,
+                                                                                                                       amount_msat: outgoing_amt_msat,
                                                                                                                        purpose,
                                                                                                                });
                                                                                                        },
@@ -3403,7 +3559,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                }
 
                for (htlc_source, payment_hash, failure_reason, destination) in failed_forwards.drain(..) {
-                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), htlc_source, &payment_hash, failure_reason, destination);
+                       self.fail_htlc_backwards_internal(htlc_source, &payment_hash, failure_reason, destination);
                }
                self.forward_htlcs(&mut phantom_receives);
 
@@ -3447,7 +3603,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                self.process_background_events();
        }
 
-       fn update_channel_fee(&self, short_to_chan_info: &mut HashMap<u64, (PublicKey, [u8; 32])>, pending_msg_events: &mut Vec<events::MessageSendEvent>, chan_id: &[u8; 32], chan: &mut Channel<Signer>, new_feerate: u32) -> (bool, NotifyOption, Result<(), MsgHandleErrInternal>) {
+       fn update_channel_fee(&self, pending_msg_events: &mut Vec<events::MessageSendEvent>, chan_id: &[u8; 32], chan: &mut Channel<<K::Target as KeysInterface>::Signer>, new_feerate: u32) -> (bool, NotifyOption, Result<(), MsgHandleErrInternal>) {
                if !chan.is_outbound() { return (true, NotifyOption::SkipPersist, Ok(())); }
                // If the feerate has decreased by less than half, don't bother
                if new_feerate <= chan.get_feerate() && new_feerate * 2 > chan.get_feerate() {
@@ -3467,30 +3623,33 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                let res = match chan.send_update_fee_and_commit(new_feerate, &self.logger) {
                        Ok(res) => Ok(res),
                        Err(e) => {
-                               let (drop, res) = convert_chan_err!(self, e, short_to_chan_info, chan, chan_id);
+                               let (drop, res) = convert_chan_err!(self, e, chan, chan_id);
                                if drop { retain_channel = false; }
                                Err(res)
                        }
                };
                let ret_err = match res {
                        Ok(Some((update_fee, commitment_signed, monitor_update))) => {
-                               if let Err(e) = self.chain_monitor.update_channel(chan.get_funding_txo().unwrap(), monitor_update) {
-                                       let (res, drop) = handle_monitor_err!(self, e, short_to_chan_info, chan, RAACommitmentOrder::CommitmentFirst, chan_id, COMMITMENT_UPDATE_ONLY);
-                                       if drop { retain_channel = false; }
-                                       res
-                               } else {
-                                       pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
-                                               node_id: chan.get_counterparty_node_id(),
-                                               updates: msgs::CommitmentUpdate {
-                                                       update_add_htlcs: Vec::new(),
-                                                       update_fulfill_htlcs: Vec::new(),
-                                                       update_fail_htlcs: Vec::new(),
-                                                       update_fail_malformed_htlcs: Vec::new(),
-                                                       update_fee: Some(update_fee),
-                                                       commitment_signed,
-                                               },
-                                       });
-                                       Ok(())
+                               match self.chain_monitor.update_channel(chan.get_funding_txo().unwrap(), monitor_update) {
+                                       ChannelMonitorUpdateStatus::Completed => {
+                                               pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
+                                                       node_id: chan.get_counterparty_node_id(),
+                                                       updates: msgs::CommitmentUpdate {
+                                                               update_add_htlcs: Vec::new(),
+                                                               update_fulfill_htlcs: Vec::new(),
+                                                               update_fail_htlcs: Vec::new(),
+                                                               update_fail_malformed_htlcs: Vec::new(),
+                                                               update_fee: Some(update_fee),
+                                                               commitment_signed,
+                                                       },
+                                               });
+                                               Ok(())
+                                       },
+                                       e => {
+                                               let (res, drop) = handle_monitor_update_res!(self, e, chan, RAACommitmentOrder::CommitmentFirst, chan_id, COMMITMENT_UPDATE_ONLY);
+                                               if drop { retain_channel = false; }
+                                               res
+                                       }
                                }
                        },
                        Ok(None) => Ok(()),
@@ -3515,9 +3674,8 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                let mut channel_state_lock = self.channel_state.lock().unwrap();
                                let channel_state = &mut *channel_state_lock;
                                let pending_msg_events = &mut channel_state.pending_msg_events;
-                               let short_to_chan_info = &mut channel_state.short_to_chan_info;
                                channel_state.by_id.retain(|chan_id, chan| {
-                                       let (retain_channel, chan_needs_persist, err) = self.update_channel_fee(short_to_chan_info, pending_msg_events, chan_id, chan, new_feerate);
+                                       let (retain_channel, chan_needs_persist, err) = self.update_channel_fee(pending_msg_events, chan_id, chan, new_feerate);
                                        if chan_needs_persist == NotifyOption::DoPersist { should_persist = NotifyOption::DoPersist; }
                                        if err.is_err() {
                                                handle_errors.push(err);
@@ -3530,6 +3688,45 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                });
        }
 
+       fn remove_stale_resolved_payments(&self) {
+               // If an outbound payment was completed, and no pending HTLCs remain, we should remove it
+               // from the map. However, if we did that immediately when the last payment HTLC is claimed,
+               // this could race the user making a duplicate send_payment call and our idempotency
+               // guarantees would be violated. Instead, we wait a few timer ticks to do the actual
+               // removal. This should be more than sufficient to ensure the idempotency of any
+               // `send_payment` calls that were made at the same time the `PaymentSent` event was being
+               // processed.
+               let mut pending_outbound_payments = self.pending_outbound_payments.lock().unwrap();
+               let pending_events = self.pending_events.lock().unwrap();
+               pending_outbound_payments.retain(|payment_id, payment| {
+                       if let PendingOutboundPayment::Fulfilled { session_privs, timer_ticks_without_htlcs, .. } = payment {
+                               let mut no_remaining_entries = session_privs.is_empty();
+                               if no_remaining_entries {
+                                       for ev in pending_events.iter() {
+                                               match ev {
+                                                       events::Event::PaymentSent { payment_id: Some(ev_payment_id), .. } |
+                                                       events::Event::PaymentPathSuccessful { payment_id: ev_payment_id, .. } |
+                                                       events::Event::PaymentPathFailed { payment_id: Some(ev_payment_id), .. } => {
+                                                               if payment_id == ev_payment_id {
+                                                                       no_remaining_entries = false;
+                                                                       break;
+                                                               }
+                                                       },
+                                                       _ => {},
+                                               }
+                                       }
+                               }
+                               if no_remaining_entries {
+                                       *timer_ticks_without_htlcs += 1;
+                                       *timer_ticks_without_htlcs <= IDEMPOTENCY_TIMEOUT_TICKS
+                               } else {
+                                       *timer_ticks_without_htlcs = 0;
+                                       true
+                               }
+                       } else { true }
+               });
+       }
+
        /// Performs actions which should happen on startup and roughly once per minute thereafter.
        ///
        /// This currently includes:
@@ -3555,10 +3752,9 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                let mut channel_state_lock = self.channel_state.lock().unwrap();
                                let channel_state = &mut *channel_state_lock;
                                let pending_msg_events = &mut channel_state.pending_msg_events;
-                               let short_to_chan_info = &mut channel_state.short_to_chan_info;
                                channel_state.by_id.retain(|chan_id, chan| {
                                        let counterparty_node_id = chan.get_counterparty_node_id();
-                                       let (retain_channel, chan_needs_persist, err) = self.update_channel_fee(short_to_chan_info, pending_msg_events, chan_id, chan, new_feerate);
+                                       let (retain_channel, chan_needs_persist, err) = self.update_channel_fee(pending_msg_events, chan_id, chan, new_feerate);
                                        if chan_needs_persist == NotifyOption::DoPersist { should_persist = NotifyOption::DoPersist; }
                                        if err.is_err() {
                                                handle_errors.push((err, counterparty_node_id));
@@ -3566,7 +3762,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        if !retain_channel { return false; }
 
                                        if let Err(e) = chan.timer_check_closing_negotiation_progress() {
-                                               let (needs_close, err) = convert_chan_err!(self, e, short_to_chan_info, chan, chan_id);
+                                               let (needs_close, err) = convert_chan_err!(self, e, chan, chan_id);
                                                handle_errors.push((Err(err), chan.get_counterparty_node_id()));
                                                if needs_close { return false; }
                                        }
@@ -3627,12 +3823,15 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                        for htlc_source in timed_out_mpp_htlcs.drain(..) {
                                let receiver = HTLCDestination::FailedPayment { payment_hash: htlc_source.1 };
-                               self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), HTLCSource::PreviousHopData(htlc_source.0.clone()), &htlc_source.1, HTLCFailReason::Reason { failure_code: 23, data: Vec::new() }, receiver );
+                               self.fail_htlc_backwards_internal(HTLCSource::PreviousHopData(htlc_source.0.clone()), &htlc_source.1, HTLCFailReason::Reason { failure_code: 23, data: Vec::new() }, receiver );
                        }
 
                        for (err, counterparty_node_id) in handle_errors.drain(..) {
                                let _ = handle_error!(self, err, counterparty_node_id);
                        }
+
+                       self.remove_stale_resolved_payments();
+
                        should_persist
                });
        }
@@ -3653,15 +3852,16 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        pub fn fail_htlc_backwards(&self, payment_hash: &PaymentHash) {
                let _persistence_guard = PersistenceNotifierGuard::notify_on_drop(&self.total_consistency_lock, &self.persistence_notifier);
 
-               let mut channel_state = Some(self.channel_state.lock().unwrap());
-               let removed_source = channel_state.as_mut().unwrap().claimable_htlcs.remove(payment_hash);
+               let removed_source = {
+                       let mut channel_state = self.channel_state.lock().unwrap();
+                       channel_state.claimable_htlcs.remove(payment_hash)
+               };
                if let Some((_, mut sources)) = removed_source {
                        for htlc in sources.drain(..) {
-                               if channel_state.is_none() { channel_state = Some(self.channel_state.lock().unwrap()); }
                                let mut htlc_msat_height_data = byte_utils::be64_to_array(htlc.value).to_vec();
                                htlc_msat_height_data.extend_from_slice(&byte_utils::be32_to_array(
                                                self.best_block.read().unwrap().height()));
-                               self.fail_htlc_backwards_internal(channel_state.take().unwrap(),
+                               self.fail_htlc_backwards_internal(
                                                HTLCSource::PreviousHopData(htlc.prev_hop), payment_hash,
                                                HTLCFailReason::Reason { failure_code: 0x4000 | 15, data: htlc_msat_height_data },
                                                HTLCDestination::FailedPayment { payment_hash: *payment_hash });
@@ -3674,7 +3874,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        ///
        /// This is for failures on the channel on which the HTLC was *received*, not failures
        /// forwarding
-       fn get_htlc_inbound_temp_fail_err_and_data(&self, desired_err_code: u16, chan: &Channel<Signer>) -> (u16, Vec<u8>) {
+       fn get_htlc_inbound_temp_fail_err_and_data(&self, desired_err_code: u16, chan: &Channel<<K::Target as KeysInterface>::Signer>) -> (u16, Vec<u8>) {
                // We can't be sure what SCID was used when relaying inbound towards us, so we have to
                // guess somewhat. If its a public channel, we figure best to just use the real SCID (as
                // we're not leaking that we have a channel with the counterparty), otherwise we try to use
@@ -3694,7 +3894,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
        /// Gets an HTLC onion failure code and error data for an `UPDATE` error, given the error code
        /// that we want to return and a channel.
-       fn get_htlc_temp_fail_err_and_data(&self, desired_err_code: u16, scid: u64, chan: &Channel<Signer>) -> (u16, Vec<u8>) {
+       fn get_htlc_temp_fail_err_and_data(&self, desired_err_code: u16, scid: u64, chan: &Channel<<K::Target as KeysInterface>::Signer>) -> (u16, Vec<u8>) {
                debug_assert_eq!(desired_err_code & 0x1000, 0x1000);
                if let Ok(upd) = self.get_channel_update_for_onion(scid, chan) {
                        let mut enc = VecWriter(Vec::with_capacity(upd.serialized_length() + 6));
@@ -3724,9 +3924,8 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                counterparty_node_id: &PublicKey
        ) {
                for (htlc_src, payment_hash) in htlcs_to_fail.drain(..) {
-                       let mut channel_state = self.channel_state.lock().unwrap();
                        let (failure_code, onion_failure_data) =
-                               match channel_state.by_id.entry(channel_id) {
+                               match self.channel_state.lock().unwrap().by_id.entry(channel_id) {
                                        hash_map::Entry::Occupied(chan_entry) => {
                                                self.get_htlc_inbound_temp_fail_err_and_data(0x1000|7, &chan_entry.get())
                                        },
@@ -3734,17 +3933,22 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                };
 
                        let receiver = HTLCDestination::NextHopChannel { node_id: Some(counterparty_node_id.clone()), channel_id };
-                       self.fail_htlc_backwards_internal(channel_state, htlc_src, &payment_hash, HTLCFailReason::Reason { failure_code, data: onion_failure_data }, receiver);
+                       self.fail_htlc_backwards_internal(htlc_src, &payment_hash, HTLCFailReason::Reason { failure_code, data: onion_failure_data }, receiver);
                }
        }
 
        /// Fails an HTLC backwards to the sender of it to us.
-       /// Note that while we take a channel_state lock as input, we do *not* assume consistency here.
-       /// There are several callsites that do stupid things like loop over a list of payment_hashes
-       /// to fail and take the channel_state lock for each iteration (as we take ownership and may
-       /// drop it). In other words, no assumptions are made that entries in claimable_htlcs point to
-       /// still-available channels.
-       fn fail_htlc_backwards_internal(&self, mut channel_state_lock: MutexGuard<ChannelHolder<Signer>>, source: HTLCSource, payment_hash: &PaymentHash, onion_error: HTLCFailReason, destination: HTLCDestination) {
+       /// Note that we do not assume that channels corresponding to failed HTLCs are still available.
+       fn fail_htlc_backwards_internal(&self, source: HTLCSource, payment_hash: &PaymentHash, onion_error: HTLCFailReason,destination: HTLCDestination) {
+               #[cfg(debug_assertions)]
+               {
+                       // Ensure that the `channel_state` lock is not held when calling this function.
+                       // This ensures that future code doesn't introduce a lock_order requirement for
+                       // `forward_htlcs` to be locked after the `channel_state` lock, which calling this
+                       // function with the `channel_state` locked would.
+                       assert!(self.channel_state.try_lock().is_ok());
+               }
+
                //TODO: There is a timing attack here where if a node fails an HTLC back to us they can
                //identify whether we sent it or not based on the (I presume) very different runtime
                //between the branches here. We should make this async and move it into the forward HTLCs
@@ -3783,7 +3987,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        log_trace!(self.logger, "Received duplicative fail for HTLC with payment_hash {}", log_bytes!(payment_hash.0));
                                        return;
                                }
-                               mem::drop(channel_state_lock);
                                let mut retry = if let Some(payment_params_data) = payment_params {
                                        let path_last_hop = path.last().expect("Outbound payments must have had a valid path");
                                        Some(RouteParameters {
@@ -3810,7 +4013,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                                }
                                                        } else {
                                                                events::Event::ProbeFailed {
-                                                                       payment_id: payment_id,
+                                                                       payment_id,
                                                                        payment_hash: payment_hash.clone(),
                                                                        path: path.clone(),
                                                                        short_channel_id,
@@ -3857,7 +4060,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                                                if self.payment_is_probe(payment_hash, &payment_id) {
                                                        events::Event::ProbeFailed {
-                                                               payment_id: payment_id,
+                                                               payment_id,
                                                                payment_hash: payment_hash.clone(),
                                                                path: path.clone(),
                                                                short_channel_id: Some(scid),
@@ -3904,10 +4107,11 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                };
 
                                let mut forward_event = None;
-                               if channel_state_lock.forward_htlcs.is_empty() {
+                               let mut forward_htlcs = self.forward_htlcs.lock().unwrap();
+                               if forward_htlcs.is_empty() {
                                        forward_event = Some(Duration::from_millis(MIN_HTLC_RELAY_HOLDING_CELL_MILLIS));
                                }
-                               match channel_state_lock.forward_htlcs.entry(short_channel_id) {
+                               match forward_htlcs.entry(short_channel_id) {
                                        hash_map::Entry::Occupied(mut entry) => {
                                                entry.get_mut().push(HTLCForwardInfo::FailHTLC { htlc_id, err_packet });
                                        },
@@ -3915,7 +4119,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                entry.insert(vec!(HTLCForwardInfo::FailHTLC { htlc_id, err_packet }));
                                        }
                                }
-                               mem::drop(channel_state_lock);
+                               mem::drop(forward_htlcs);
                                let mut pending_events = self.pending_events.lock().unwrap();
                                if let Some(time) = forward_event {
                                        pending_events.push(events::Event::PendingHTLCsForwardable {
@@ -3953,8 +4157,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                let _persistence_guard = PersistenceNotifierGuard::notify_on_drop(&self.total_consistency_lock, &self.persistence_notifier);
 
-               let mut channel_state = Some(self.channel_state.lock().unwrap());
-               let removed_source = channel_state.as_mut().unwrap().claimable_htlcs.remove(&payment_hash);
+               let removed_source = self.channel_state.lock().unwrap().claimable_htlcs.remove(&payment_hash);
                if let Some((payment_purpose, mut sources)) = removed_source {
                        assert!(!sources.is_empty());
 
@@ -3972,11 +4175,24 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        let mut claimable_amt_msat = 0;
                        let mut expected_amt_msat = None;
                        let mut valid_mpp = true;
+                       let mut errs = Vec::new();
+                       let mut claimed_any_htlcs = false;
+                       let mut channel_state_lock = self.channel_state.lock().unwrap();
+                       let channel_state = &mut *channel_state_lock;
                        for htlc in sources.iter() {
-                               if let None = channel_state.as_ref().unwrap().short_to_chan_info.get(&htlc.prev_hop.short_channel_id) {
+                               let chan_id = match self.short_to_chan_info.read().unwrap().get(&htlc.prev_hop.short_channel_id) {
+                                       Some((_cp_id, chan_id)) => chan_id.clone(),
+                                       None => {
+                                               valid_mpp = false;
+                                               break;
+                                       }
+                               };
+
+                               if let None = channel_state.by_id.get(&chan_id) {
                                        valid_mpp = false;
                                        break;
                                }
+
                                if expected_amt_msat.is_some() && expected_amt_msat != Some(htlc.total_msat) {
                                        log_error!(self.logger, "Somehow ended up with an MPP payment with different total amounts - this should not be reachable!");
                                        debug_assert!(false);
@@ -4006,21 +4222,9 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        expected_amt_msat.unwrap(), claimable_amt_msat);
                                return;
                        }
-
-                       let mut errs = Vec::new();
-                       let mut claimed_any_htlcs = false;
-                       for htlc in sources.drain(..) {
-                               if !valid_mpp {
-                                       if channel_state.is_none() { channel_state = Some(self.channel_state.lock().unwrap()); }
-                                       let mut htlc_msat_height_data = byte_utils::be64_to_array(htlc.value).to_vec();
-                                       htlc_msat_height_data.extend_from_slice(&byte_utils::be32_to_array(
-                                                       self.best_block.read().unwrap().height()));
-                                       self.fail_htlc_backwards_internal(channel_state.take().unwrap(),
-                                                                        HTLCSource::PreviousHopData(htlc.prev_hop), &payment_hash,
-                                                                        HTLCFailReason::Reason { failure_code: 0x4000|15, data: htlc_msat_height_data },
-                                                                        HTLCDestination::FailedPayment { payment_hash } );
-                               } else {
-                                       match self.claim_funds_from_hop(channel_state.as_mut().unwrap(), htlc.prev_hop, payment_preimage) {
+                       if valid_mpp {
+                               for htlc in sources.drain(..) {
+                                       match self.claim_funds_from_hop(&mut channel_state_lock, htlc.prev_hop, payment_preimage) {
                                                ClaimFundsFromHop::MonitorUpdateFail(pk, err, _) => {
                                                        if let msgs::ErrorAction::IgnoreError = err.err.action {
                                                                // We got a temporary failure updating monitor, but will claim the
@@ -4040,6 +4244,18 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        }
                                }
                        }
+                       mem::drop(channel_state_lock);
+                       if !valid_mpp {
+                               for htlc in sources.drain(..) {
+                                       let mut htlc_msat_height_data = byte_utils::be64_to_array(htlc.value).to_vec();
+                                       htlc_msat_height_data.extend_from_slice(&byte_utils::be32_to_array(
+                                               self.best_block.read().unwrap().height()));
+                                       self.fail_htlc_backwards_internal(
+                                               HTLCSource::PreviousHopData(htlc.prev_hop), &payment_hash,
+                                               HTLCFailReason::Reason { failure_code: 0x4000|15, data: htlc_msat_height_data },
+                                               HTLCDestination::FailedPayment { payment_hash } );
+                               }
+                       }
 
                        if claimed_any_htlcs {
                                self.pending_events.lock().unwrap().push(events::Event::PaymentClaimed {
@@ -4049,10 +4265,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                });
                        }
 
-                       // Now that we've done the entire above loop in one lock, we can handle any errors
-                       // which were generated.
-                       channel_state.take();
-
+                       // Now we can handle any errors which were generated.
                        for (counterparty_node_id, err) in errs.drain(..) {
                                let res: Result<(), _> = Err(err);
                                let _ = handle_error!(self, res, counterparty_node_id);
@@ -4060,10 +4273,10 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                }
        }
 
-       fn claim_funds_from_hop(&self, channel_state_lock: &mut MutexGuard<ChannelHolder<Signer>>, prev_hop: HTLCPreviousHopData, payment_preimage: PaymentPreimage) -> ClaimFundsFromHop {
+       fn claim_funds_from_hop(&self, channel_state_lock: &mut MutexGuard<ChannelHolder<<K::Target as KeysInterface>::Signer>>, prev_hop: HTLCPreviousHopData, payment_preimage: PaymentPreimage) -> ClaimFundsFromHop {
                //TODO: Delay the claimed_funds relaying just like we do outbound relay!
                let channel_state = &mut **channel_state_lock;
-               let chan_id = match channel_state.short_to_chan_info.get(&prev_hop.short_channel_id) {
+               let chan_id = match self.short_to_chan_info.read().unwrap().get(&prev_hop.short_channel_id) {
                        Some((_cp_id, chan_id)) => chan_id.clone(),
                        None => {
                                return ClaimFundsFromHop::PrevHopForceClosed
@@ -4074,15 +4287,18 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        match chan.get_mut().get_update_fulfill_htlc_and_commit(prev_hop.htlc_id, payment_preimage, &self.logger) {
                                Ok(msgs_monitor_option) => {
                                        if let UpdateFulfillCommitFetch::NewClaim { msgs, htlc_value_msat, monitor_update } = msgs_monitor_option {
-                                               if let Err(e) = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
-                                                       log_given_level!(self.logger, if e == ChannelMonitorUpdateErr::PermanentFailure { Level::Error } else { Level::Debug },
-                                                               "Failed to update channel monitor with preimage {:?}: {:?}",
-                                                               payment_preimage, e);
-                                                       return ClaimFundsFromHop::MonitorUpdateFail(
-                                                               chan.get().get_counterparty_node_id(),
-                                                               handle_monitor_err!(self, e, channel_state, chan, RAACommitmentOrder::CommitmentFirst, false, msgs.is_some()).unwrap_err(),
-                                                               Some(htlc_value_msat)
-                                                       );
+                                               match self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
+                                                       ChannelMonitorUpdateStatus::Completed => {},
+                                                       e => {
+                                                               log_given_level!(self.logger, if e == ChannelMonitorUpdateStatus::PermanentFailure { Level::Error } else { Level::Debug },
+                                                                       "Failed to update channel monitor with preimage {:?}: {:?}",
+                                                                       payment_preimage, e);
+                                                               return ClaimFundsFromHop::MonitorUpdateFail(
+                                                                       chan.get().get_counterparty_node_id(),
+                                                                       handle_monitor_update_res!(self, e, chan, RAACommitmentOrder::CommitmentFirst, false, msgs.is_some()).unwrap_err(),
+                                                                       Some(htlc_value_msat)
+                                                               );
+                                                       }
                                                }
                                                if let Some((msg, commitment_signed)) = msgs {
                                                        log_debug!(self.logger, "Claiming funds for HTLC with preimage {} resulted in a commitment_signed for channel {}",
@@ -4105,20 +4321,23 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        }
                                },
                                Err((e, monitor_update)) => {
-                                       if let Err(e) = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
-                                               log_given_level!(self.logger, if e == ChannelMonitorUpdateErr::PermanentFailure { Level::Error } else { Level::Info },
-                                                       "Failed to update channel monitor with preimage {:?} immediately prior to force-close: {:?}",
-                                                       payment_preimage, e);
+                                       match self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
+                                               ChannelMonitorUpdateStatus::Completed => {},
+                                               e => {
+                                                       log_given_level!(self.logger, if e == ChannelMonitorUpdateStatus::PermanentFailure { Level::Error } else { Level::Info },
+                                                               "Failed to update channel monitor with preimage {:?} immediately prior to force-close: {:?}",
+                                                               payment_preimage, e);
+                                               },
                                        }
                                        let counterparty_node_id = chan.get().get_counterparty_node_id();
-                                       let (drop, res) = convert_chan_err!(self, e, channel_state.short_to_chan_info, chan.get_mut(), &chan_id);
+                                       let (drop, res) = convert_chan_err!(self, e, chan.get_mut(), &chan_id);
                                        if drop {
                                                chan.remove_entry();
                                        }
                                        return ClaimFundsFromHop::MonitorUpdateFail(counterparty_node_id, res, None);
                                },
                        }
-               } else { unreachable!(); }
+               } else { return ClaimFundsFromHop::PrevHopForceClosed }
        }
 
        fn finalize_claims(&self, mut sources: Vec<HTLCSource>) {
@@ -4139,15 +4358,12 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                        }
                                                );
                                        }
-                                       if payment.get().remaining_parts() == 0 {
-                                               payment.remove();
-                                       }
                                }
                        }
                }
        }
 
-       fn claim_funds_internal(&self, mut channel_state_lock: MutexGuard<ChannelHolder<Signer>>, source: HTLCSource, payment_preimage: PaymentPreimage, forwarded_htlc_value_msat: Option<u64>, from_onchain: bool, next_channel_id: [u8; 32]) {
+       fn claim_funds_internal(&self, mut channel_state_lock: MutexGuard<ChannelHolder<<K::Target as KeysInterface>::Signer>>, source: HTLCSource, payment_preimage: PaymentPreimage, forwarded_htlc_value_msat: Option<u64>, from_onchain: bool, next_channel_id: [u8; 32]) {
                match source {
                        HTLCSource::OutboundRoute { session_priv, payment_id, path, .. } => {
                                mem::drop(channel_state_lock);
@@ -4187,10 +4403,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                                }
                                                        );
                                                }
-
-                                               if payment.get().remaining_parts() == 0 {
-                                                       payment.remove();
-                                               }
                                        }
                                } else {
                                        log_trace!(self.logger, "Received duplicative fulfill for HTLC with payment_preimage {}", log_bytes!(payment_preimage.0));
@@ -4215,9 +4427,14 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        // We update the ChannelMonitor on the backward link, after
                                        // receiving an offchain preimage event from the forward link (the
                                        // event being update_fulfill_htlc).
-                                       if let Err(e) = self.chain_monitor.update_channel(prev_outpoint, preimage_update) {
+                                       let update_res = self.chain_monitor.update_channel(prev_outpoint, preimage_update);
+                                       if update_res != ChannelMonitorUpdateStatus::Completed {
+                                               // TODO: This needs to be handled somehow - if we receive a monitor update
+                                               // with a preimage we *must* somehow manage to propagate it to the upstream
+                                               // channel, or we must have an ability to receive the same event and try
+                                               // again on restart.
                                                log_error!(self.logger, "Critical error: failed to update channel monitor with preimage {:?}: {:?}",
-                                                                                        payment_preimage, e);
+                                                       payment_preimage, update_res);
                                        }
                                        // Note that we do *not* set `claimed_htlc` to false here. In fact, this
                                        // totally could be a duplicate claim, but we have no way of knowing
@@ -4299,7 +4516,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                self.finalize_claims(finalized_claims);
                for failure in pending_failures.drain(..) {
                        let receiver = HTLCDestination::NextHopChannel { node_id: Some(counterparty_node_id), channel_id: funding_txo.to_channel_id() };
-                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), failure.0, &failure.1, failure.2, receiver);
+                       self.fail_htlc_backwards_internal(failure.0, &failure.1, failure.2, receiver);
                }
        }
 
@@ -4368,7 +4585,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                }
                                        };
                                        channel_state.pending_msg_events.push(send_msg_err_event);
-                                       let _ = remove_channel!(self, channel_state, channel);
+                                       let _ = remove_channel!(self, channel);
                                        return Err(APIError::APIMisuseError { err: "Please use accept_inbound_channel_from_trusted_peer_0conf to accept channels with zero confirmations.".to_owned() });
                                }
 
@@ -4448,7 +4665,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        if chan.get().get_counterparty_node_id() != *counterparty_node_id {
                                                return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.temporary_channel_id));
                                        }
-                                       try_chan_entry!(self, chan.get_mut().accept_channel(&msg, &self.default_configuration.channel_handshake_limits, &their_features), channel_state, chan);
+                                       try_chan_entry!(self, chan.get_mut().accept_channel(&msg, &self.default_configuration.channel_handshake_limits, &their_features), chan);
                                        (chan.get().get_value_satoshis(), chan.get().get_funding_redeemscript().to_v0_p2wsh(), chan.get().get_user_id())
                                },
                                hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.temporary_channel_id))
@@ -4475,36 +4692,35 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        if chan.get().get_counterparty_node_id() != *counterparty_node_id {
                                                return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.temporary_channel_id));
                                        }
-                                       (try_chan_entry!(self, chan.get_mut().funding_created(msg, best_block, &self.logger), channel_state, chan), chan.remove())
+                                       (try_chan_entry!(self, chan.get_mut().funding_created(msg, best_block, &self.logger), chan), chan.remove())
                                },
                                hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.temporary_channel_id))
                        }
                };
                // Because we have exclusive ownership of the channel here we can release the channel_state
                // lock before watch_channel
-               if let Err(e) = self.chain_monitor.watch_channel(monitor.get_funding_txo().0, monitor) {
-                       match e {
-                               ChannelMonitorUpdateErr::PermanentFailure => {
-                                       // Note that we reply with the new channel_id in error messages if we gave up on the
-                                       // channel, not the temporary_channel_id. This is compatible with ourselves, but the
-                                       // spec is somewhat ambiguous here. Not a huge deal since we'll send error messages for
-                                       // any messages referencing a previously-closed channel anyway.
-                                       // We do not do a force-close here as that would generate a monitor update for
-                                       // a monitor that we didn't manage to store (and that we don't care about - we
-                                       // don't respond with the funding_signed so the channel can never go on chain).
-                                       let (_monitor_update, failed_htlcs) = chan.force_shutdown(false);
-                                       assert!(failed_htlcs.is_empty());
-                                       return Err(MsgHandleErrInternal::send_err_msg_no_close("ChannelMonitor storage failure".to_owned(), funding_msg.channel_id));
-                               },
-                               ChannelMonitorUpdateErr::TemporaryFailure => {
-                                       // There's no problem signing a counterparty's funding transaction if our monitor
-                                       // hasn't persisted to disk yet - we can't lose money on a transaction that we haven't
-                                       // accepted payment from yet. We do, however, need to wait to send our channel_ready
-                                       // until we have persisted our monitor.
-                                       chan.monitor_update_failed(false, false, channel_ready.is_some(), Vec::new(), Vec::new(), Vec::new());
-                                       channel_ready = None; // Don't send the channel_ready now
-                               },
-                       }
+               match self.chain_monitor.watch_channel(monitor.get_funding_txo().0, monitor) {
+                       ChannelMonitorUpdateStatus::Completed => {},
+                       ChannelMonitorUpdateStatus::PermanentFailure => {
+                               // Note that we reply with the new channel_id in error messages if we gave up on the
+                               // channel, not the temporary_channel_id. This is compatible with ourselves, but the
+                               // spec is somewhat ambiguous here. Not a huge deal since we'll send error messages for
+                               // any messages referencing a previously-closed channel anyway.
+                               // We do not propagate the monitor update to the user as it would be for a monitor
+                               // that we didn't manage to store (and that we don't care about - we don't respond
+                               // with the funding_signed so the channel can never go on chain).
+                               let (_monitor_update, failed_htlcs) = chan.force_shutdown(false);
+                               assert!(failed_htlcs.is_empty());
+                               return Err(MsgHandleErrInternal::send_err_msg_no_close("ChannelMonitor storage failure".to_owned(), funding_msg.channel_id));
+                       },
+                       ChannelMonitorUpdateStatus::InProgress => {
+                               // There's no problem signing a counterparty's funding transaction if our monitor
+                               // hasn't persisted to disk yet - we can't lose money on a transaction that we haven't
+                               // accepted payment from yet. We do, however, need to wait to send our channel_ready
+                               // until we have persisted our monitor.
+                               chan.monitor_updating_paused(false, false, channel_ready.is_some(), Vec::new(), Vec::new(), Vec::new());
+                               channel_ready = None; // Don't send the channel_ready now
+                       },
                }
                let mut channel_state_lock = self.channel_state.lock().unwrap();
                let channel_state = &mut *channel_state_lock;
@@ -4529,7 +4745,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        msg: funding_msg,
                                });
                                if let Some(msg) = channel_ready {
-                                       send_channel_ready!(channel_state.short_to_chan_info, channel_state.pending_msg_events, chan, msg);
+                                       send_channel_ready!(self, channel_state.pending_msg_events, chan, msg);
                                }
                                e.insert(chan);
                        }
@@ -4549,22 +4765,25 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        }
                                        let (monitor, funding_tx, channel_ready) = match chan.get_mut().funding_signed(&msg, best_block, &self.logger) {
                                                Ok(update) => update,
-                                               Err(e) => try_chan_entry!(self, Err(e), channel_state, chan),
+                                               Err(e) => try_chan_entry!(self, Err(e), chan),
                                        };
-                                       if let Err(e) = self.chain_monitor.watch_channel(chan.get().get_funding_txo().unwrap(), monitor) {
-                                               let mut res = handle_monitor_err!(self, e, channel_state, chan, RAACommitmentOrder::RevokeAndACKFirst, channel_ready.is_some(), OPTIONALLY_RESEND_FUNDING_LOCKED);
-                                               if let Err(MsgHandleErrInternal { ref mut shutdown_finish, .. }) = res {
-                                                       // We weren't able to watch the channel to begin with, so no updates should be made on
-                                                       // it. Previously, full_stack_target found an (unreachable) panic when the
-                                                       // monitor update contained within `shutdown_finish` was applied.
-                                                       if let Some((ref mut shutdown_finish, _)) = shutdown_finish {
-                                                               shutdown_finish.0.take();
+                                       match self.chain_monitor.watch_channel(chan.get().get_funding_txo().unwrap(), monitor) {
+                                               ChannelMonitorUpdateStatus::Completed => {},
+                                               e => {
+                                                       let mut res = handle_monitor_update_res!(self, e, chan, RAACommitmentOrder::RevokeAndACKFirst, channel_ready.is_some(), OPTIONALLY_RESEND_FUNDING_LOCKED);
+                                                       if let Err(MsgHandleErrInternal { ref mut shutdown_finish, .. }) = res {
+                                                               // We weren't able to watch the channel to begin with, so no updates should be made on
+                                                               // it. Previously, full_stack_target found an (unreachable) panic when the
+                                                               // monitor update contained within `shutdown_finish` was applied.
+                                                               if let Some((ref mut shutdown_finish, _)) = shutdown_finish {
+                                                                       shutdown_finish.0.take();
+                                                               }
                                                        }
-                                               }
-                                               return res
+                                                       return res
+                                               },
                                        }
                                        if let Some(msg) = channel_ready {
-                                               send_channel_ready!(channel_state.short_to_chan_info, channel_state.pending_msg_events, chan.get(), msg);
+                                               send_channel_ready!(self, channel_state.pending_msg_events, chan.get(), msg);
                                        }
                                        funding_tx
                                },
@@ -4585,7 +4804,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                }
                                let announcement_sigs_opt = try_chan_entry!(self, chan.get_mut().channel_ready(&msg, self.get_our_node_id(),
-                                       self.genesis_hash.clone(), &self.best_block.read().unwrap(), &self.logger), channel_state, chan);
+                                       self.genesis_hash.clone(), &self.best_block.read().unwrap(), &self.logger), chan);
                                if let Some(announcement_sigs) = announcement_sigs_opt {
                                        log_trace!(self.logger, "Sending announcement_signatures for channel {}", log_bytes!(chan.get().channel_id()));
                                        channel_state.pending_msg_events.push(events::MessageSendEvent::SendAnnouncementSignatures {
@@ -4606,6 +4825,9 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                });
                                        }
                                }
+
+                               emit_channel_ready_event!(self, chan.get_mut());
+
                                Ok(())
                        },
                        hash_map::Entry::Vacant(_) => Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
@@ -4630,18 +4852,17 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                        if chan_entry.get().sent_shutdown() { " after we initiated shutdown" } else { "" });
                                        }
 
-                                       let (shutdown, monitor_update, htlcs) = try_chan_entry!(self, chan_entry.get_mut().shutdown(&self.keys_manager, &their_features, &msg), channel_state, chan_entry);
+                                       let (shutdown, monitor_update, htlcs) = try_chan_entry!(self, chan_entry.get_mut().shutdown(&self.keys_manager, &their_features, &msg), chan_entry);
                                        dropped_htlcs = htlcs;
 
                                        // Update the monitor with the shutdown script if necessary.
                                        if let Some(monitor_update) = monitor_update {
-                                               if let Err(e) = self.chain_monitor.update_channel(chan_entry.get().get_funding_txo().unwrap(), monitor_update) {
-                                                       let (result, is_permanent) =
-                                                               handle_monitor_err!(self, e, channel_state.short_to_chan_info, chan_entry.get_mut(), RAACommitmentOrder::CommitmentFirst, chan_entry.key(), NO_UPDATE);
-                                                       if is_permanent {
-                                                               remove_channel!(self, channel_state, chan_entry);
-                                                               break result;
-                                                       }
+                                               let update_res = self.chain_monitor.update_channel(chan_entry.get().get_funding_txo().unwrap(), monitor_update);
+                                               let (result, is_permanent) =
+                                                       handle_monitor_update_res!(self, update_res, chan_entry.get_mut(), RAACommitmentOrder::CommitmentFirst, chan_entry.key(), NO_UPDATE);
+                                               if is_permanent {
+                                                       remove_channel!(self, chan_entry);
+                                                       break result;
                                                }
                                        }
 
@@ -4659,7 +4880,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                };
                for htlc_source in dropped_htlcs.drain(..) {
                        let receiver = HTLCDestination::NextHopChannel { node_id: Some(counterparty_node_id.clone()), channel_id: msg.channel_id };
-                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), htlc_source.0, &htlc_source.1, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
+                       self.fail_htlc_backwards_internal(htlc_source.0, &htlc_source.1, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
                }
 
                let _ = handle_error!(self, result, *counterparty_node_id);
@@ -4675,7 +4896,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        if chan_entry.get().get_counterparty_node_id() != *counterparty_node_id {
                                                return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                        }
-                                       let (closing_signed, tx) = try_chan_entry!(self, chan_entry.get_mut().closing_signed(&self.fee_estimator, &msg), channel_state, chan_entry);
+                                       let (closing_signed, tx) = try_chan_entry!(self, chan_entry.get_mut().closing_signed(&self.fee_estimator, &msg), chan_entry);
                                        if let Some(msg) = closing_signed {
                                                channel_state.pending_msg_events.push(events::MessageSendEvent::SendClosingSigned {
                                                        node_id: counterparty_node_id.clone(),
@@ -4688,7 +4909,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                // also implies there are no pending HTLCs left on the channel, so we can
                                                // fully delete it from tracking (the channel monitor is still around to
                                                // watch for old state broadcasts)!
-                                               (tx, Some(remove_channel!(self, channel_state, chan_entry)))
+                                               (tx, Some(remove_channel!(self, chan_entry)))
                                        } else { (tx, None) }
                                },
                                hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
@@ -4730,7 +4951,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                }
 
-                               let create_pending_htlc_status = |chan: &Channel<Signer>, pending_forward_info: PendingHTLCStatus, error_code: u16| {
+                               let create_pending_htlc_status = |chan: &Channel<<K::Target as KeysInterface>::Signer>, pending_forward_info: PendingHTLCStatus, error_code: u16| {
                                        // If the update_add is completely bogus, the call will Err and we will close,
                                        // but if we've sent a shutdown and they haven't acknowledged it yet, we just
                                        // want to reject the new HTLC and fail it backwards instead of forwarding.
@@ -4752,7 +4973,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                _ => pending_forward_info
                                        }
                                };
-                               try_chan_entry!(self, chan.get_mut().update_add_htlc(&msg, pending_forward_info, create_pending_htlc_status, &self.logger), channel_state, chan);
+                               try_chan_entry!(self, chan.get_mut().update_add_htlc(&msg, pending_forward_info, create_pending_htlc_status, &self.logger), chan);
                        },
                        hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
                }
@@ -4768,7 +4989,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        if chan.get().get_counterparty_node_id() != *counterparty_node_id {
                                                return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                        }
-                                       try_chan_entry!(self, chan.get_mut().update_fulfill_htlc(&msg), channel_state, chan)
+                                       try_chan_entry!(self, chan.get_mut().update_fulfill_htlc(&msg), chan)
                                },
                                hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
                        }
@@ -4785,7 +5006,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                if chan.get().get_counterparty_node_id() != *counterparty_node_id {
                                        return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                }
-                               try_chan_entry!(self, chan.get_mut().update_fail_htlc(&msg, HTLCFailReason::LightningError { err: msg.reason.clone() }), channel_state, chan);
+                               try_chan_entry!(self, chan.get_mut().update_fail_htlc(&msg, HTLCFailReason::LightningError { err: msg.reason.clone() }), chan);
                        },
                        hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
                }
@@ -4802,9 +5023,9 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                }
                                if (msg.failure_code & 0x8000) == 0 {
                                        let chan_err: ChannelError = ChannelError::Close("Got update_fail_malformed_htlc with BADONION not set".to_owned());
-                                       try_chan_entry!(self, Err(chan_err), channel_state, chan);
+                                       try_chan_entry!(self, Err(chan_err), chan);
                                }
-                               try_chan_entry!(self, chan.get_mut().update_fail_malformed_htlc(&msg, HTLCFailReason::Reason { failure_code: msg.failure_code, data: Vec::new() }), channel_state, chan);
+                               try_chan_entry!(self, chan.get_mut().update_fail_malformed_htlc(&msg, HTLCFailReason::Reason { failure_code: msg.failure_code, data: Vec::new() }), chan);
                                Ok(())
                        },
                        hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
@@ -4821,18 +5042,20 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                }
                                let (revoke_and_ack, commitment_signed, monitor_update) =
                                        match chan.get_mut().commitment_signed(&msg, &self.logger) {
-                                               Err((None, e)) => try_chan_entry!(self, Err(e), channel_state, chan),
+                                               Err((None, e)) => try_chan_entry!(self, Err(e), chan),
                                                Err((Some(update), e)) => {
                                                        assert!(chan.get().is_awaiting_monitor_update());
                                                        let _ = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), update);
-                                                       try_chan_entry!(self, Err(e), channel_state, chan);
+                                                       try_chan_entry!(self, Err(e), chan);
                                                        unreachable!();
                                                },
                                                Ok(res) => res
                                        };
-                               if let Err(e) = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update) {
-                                       return_monitor_err!(self, e, channel_state, chan, RAACommitmentOrder::RevokeAndACKFirst, true, commitment_signed.is_some());
+                               let update_res = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), monitor_update);
+                               if let Err(e) = handle_monitor_update_res!(self, update_res, chan, RAACommitmentOrder::RevokeAndACKFirst, true, commitment_signed.is_some()) {
+                                       return Err(e);
                                }
+
                                channel_state.pending_msg_events.push(events::MessageSendEvent::SendRevokeAndACK {
                                        node_id: counterparty_node_id.clone(),
                                        msg: revoke_and_ack,
@@ -4861,23 +5084,23 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                for &mut (prev_short_channel_id, prev_funding_outpoint, ref mut pending_forwards) in per_source_pending_forwards {
                        let mut forward_event = None;
                        if !pending_forwards.is_empty() {
-                               let mut channel_state = self.channel_state.lock().unwrap();
-                               if channel_state.forward_htlcs.is_empty() {
+                               let mut forward_htlcs = self.forward_htlcs.lock().unwrap();
+                               if forward_htlcs.is_empty() {
                                        forward_event = Some(Duration::from_millis(MIN_HTLC_RELAY_HOLDING_CELL_MILLIS))
                                }
                                for (forward_info, prev_htlc_id) in pending_forwards.drain(..) {
-                                       match channel_state.forward_htlcs.entry(match forward_info.routing {
+                                       match forward_htlcs.entry(match forward_info.routing {
                                                        PendingHTLCRouting::Forward { short_channel_id, .. } => short_channel_id,
                                                        PendingHTLCRouting::Receive { .. } => 0,
                                                        PendingHTLCRouting::ReceiveKeysend { .. } => 0,
                                        }) {
                                                hash_map::Entry::Occupied(mut entry) => {
-                                                       entry.get_mut().push(HTLCForwardInfo::AddHTLC { prev_short_channel_id, prev_funding_outpoint,
-                                                                                                       prev_htlc_id, forward_info });
+                                                       entry.get_mut().push(HTLCForwardInfo::AddHTLC(PendingAddHTLCInfo {
+                                                               prev_short_channel_id, prev_funding_outpoint, prev_htlc_id, forward_info }));
                                                },
                                                hash_map::Entry::Vacant(entry) => {
-                                                       entry.insert(vec!(HTLCForwardInfo::AddHTLC { prev_short_channel_id, prev_funding_outpoint,
-                                                                                                    prev_htlc_id, forward_info }));
+                                                       entry.insert(vec!(HTLCForwardInfo::AddHTLC(PendingAddHTLCInfo {
+                                                               prev_short_channel_id, prev_funding_outpoint, prev_htlc_id, forward_info })));
                                                }
                                        }
                                }
@@ -4904,26 +5127,27 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        if chan.get().get_counterparty_node_id() != *counterparty_node_id {
                                                break Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                        }
-                                       let was_frozen_for_monitor = chan.get().is_awaiting_monitor_update();
+                                       let was_paused_for_mon_update = chan.get().is_awaiting_monitor_update();
                                        let raa_updates = break_chan_entry!(self,
-                                               chan.get_mut().revoke_and_ack(&msg, &self.logger), channel_state, chan);
+                                               chan.get_mut().revoke_and_ack(&msg, &self.logger), chan);
                                        htlcs_to_fail = raa_updates.holding_cell_failed_htlcs;
-                                       if let Err(e) = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), raa_updates.monitor_update) {
-                                               if was_frozen_for_monitor {
-                                                       assert!(raa_updates.commitment_update.is_none());
-                                                       assert!(raa_updates.accepted_htlcs.is_empty());
-                                                       assert!(raa_updates.failed_htlcs.is_empty());
-                                                       assert!(raa_updates.finalized_claimed_htlcs.is_empty());
-                                                       break Err(MsgHandleErrInternal::ignore_no_close("Previous monitor update failure prevented responses to RAA".to_owned()));
-                                               } else {
-                                                       if let Err(e) = handle_monitor_err!(self, e, channel_state, chan,
-                                                                       RAACommitmentOrder::CommitmentFirst, false,
-                                                                       raa_updates.commitment_update.is_some(), false,
-                                                                       raa_updates.accepted_htlcs, raa_updates.failed_htlcs,
-                                                                       raa_updates.finalized_claimed_htlcs) {
-                                                               break Err(e);
-                                                       } else { unreachable!(); }
-                                               }
+                                       let update_res = self.chain_monitor.update_channel(chan.get().get_funding_txo().unwrap(), raa_updates.monitor_update);
+                                       if was_paused_for_mon_update {
+                                               assert!(update_res != ChannelMonitorUpdateStatus::Completed);
+                                               assert!(raa_updates.commitment_update.is_none());
+                                               assert!(raa_updates.accepted_htlcs.is_empty());
+                                               assert!(raa_updates.failed_htlcs.is_empty());
+                                               assert!(raa_updates.finalized_claimed_htlcs.is_empty());
+                                               break Err(MsgHandleErrInternal::ignore_no_close("Existing pending monitor update prevented responses to RAA".to_owned()));
+                                       }
+                                       if update_res != ChannelMonitorUpdateStatus::Completed {
+                                               if let Err(e) = handle_monitor_update_res!(self, update_res, chan,
+                                                               RAACommitmentOrder::CommitmentFirst, false,
+                                                               raa_updates.commitment_update.is_some(), false,
+                                                               raa_updates.accepted_htlcs, raa_updates.failed_htlcs,
+                                                               raa_updates.finalized_claimed_htlcs) {
+                                                       break Err(e);
+                                               } else { unreachable!(); }
                                        }
                                        if let Some(updates) = raa_updates.commitment_update {
                                                channel_state.pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
@@ -4947,7 +5171,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        {
                                for failure in pending_failures.drain(..) {
                                        let receiver = HTLCDestination::NextHopChannel { node_id: Some(*counterparty_node_id), channel_id: channel_outpoint.to_channel_id() };
-                                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), failure.0, &failure.1, failure.2, receiver);
+                                       self.fail_htlc_backwards_internal(failure.0, &failure.1, failure.2, receiver);
                                }
                                self.forward_htlcs(&mut [(short_channel_id, channel_outpoint, pending_forwards)]);
                                self.finalize_claims(finalized_claim_htlcs);
@@ -4965,7 +5189,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                if chan.get().get_counterparty_node_id() != *counterparty_node_id {
                                        return Err(MsgHandleErrInternal::send_err_msg_no_close("Got a message for a channel from the wrong node!".to_owned(), msg.channel_id));
                                }
-                               try_chan_entry!(self, chan.get_mut().update_fee(&self.fee_estimator, &msg), channel_state, chan);
+                               try_chan_entry!(self, chan.get_mut().update_fee(&self.fee_estimator, &msg), chan);
                        },
                        hash_map::Entry::Vacant(_) => return Err(MsgHandleErrInternal::send_err_msg_no_close("Failed to find corresponding channel".to_owned(), msg.channel_id))
                }
@@ -4987,7 +5211,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                                channel_state.pending_msg_events.push(events::MessageSendEvent::BroadcastChannelAnnouncement {
                                        msg: try_chan_entry!(self, chan.get_mut().announcement_signatures(
-                                               self.get_our_node_id(), self.genesis_hash.clone(), self.best_block.read().unwrap().height(), msg), channel_state, chan),
+                                               self.get_our_node_id(), self.genesis_hash.clone(), self.best_block.read().unwrap().height(), msg), chan),
                                        // Note that announcement_signatures fails if the channel cannot be announced,
                                        // so get_channel_update_for_broadcast will never fail by the time we get here.
                                        update_msg: self.get_channel_update_for_broadcast(chan.get()).unwrap(),
@@ -5000,15 +5224,15 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
        /// Returns ShouldPersist if anything changed, otherwise either SkipPersist or an Err.
        fn internal_channel_update(&self, counterparty_node_id: &PublicKey, msg: &msgs::ChannelUpdate) -> Result<NotifyOption, MsgHandleErrInternal> {
-               let mut channel_state_lock = self.channel_state.lock().unwrap();
-               let channel_state = &mut *channel_state_lock;
-               let chan_id = match channel_state.short_to_chan_info.get(&msg.contents.short_channel_id) {
+               let chan_id = match self.short_to_chan_info.read().unwrap().get(&msg.contents.short_channel_id) {
                        Some((_cp_id, chan_id)) => chan_id.clone(),
                        None => {
                                // It's not a local channel
                                return Ok(NotifyOption::SkipPersist)
                        }
                };
+               let mut channel_state_lock = self.channel_state.lock().unwrap();
+               let channel_state = &mut *channel_state_lock;
                match channel_state.by_id.entry(chan_id) {
                        hash_map::Entry::Occupied(mut chan) => {
                                if chan.get().get_counterparty_node_id() != *counterparty_node_id {
@@ -5026,10 +5250,10 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        return Ok(NotifyOption::SkipPersist);
                                } else {
                                        log_debug!(self.logger, "Received channel_update for channel {}.", log_bytes!(chan_id));
-                                       try_chan_entry!(self, chan.get_mut().channel_update(&msg), channel_state, chan);
+                                       try_chan_entry!(self, chan.get_mut().channel_update(&msg), chan);
                                }
                        },
-                       hash_map::Entry::Vacant(_) => unreachable!()
+                       hash_map::Entry::Vacant(_) => return Ok(NotifyOption::SkipPersist)
                }
                Ok(NotifyOption::DoPersist)
        }
@@ -5051,7 +5275,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                        // add-HTLCs on disconnect, we may be handed HTLCs to fail backwards here.
                                        let responses = try_chan_entry!(self, chan.get_mut().channel_reestablish(
                                                msg, &self.logger, self.our_network_pubkey.clone(), self.genesis_hash,
-                                               &*self.best_block.read().unwrap()), channel_state, chan);
+                                               &*self.best_block.read().unwrap()), chan);
                                        let mut channel_update = None;
                                        if let Some(msg) = responses.shutdown_msg {
                                                channel_state.pending_msg_events.push(events::MessageSendEvent::SendShutdown {
@@ -5105,7 +5329,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                } else {
                                                        log_trace!(self.logger, "Failing HTLC with hash {} from our monitor", log_bytes!(htlc_update.payment_hash.0));
                                                        let receiver = HTLCDestination::NextHopChannel { node_id: counterparty_node_id, channel_id: funding_outpoint.to_channel_id() };
-                                                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), htlc_update.source, &htlc_update.payment_hash, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
+                                                       self.fail_htlc_backwards_internal(htlc_update.source, &htlc_update.payment_hash, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
                                                }
                                        },
                                        MonitorEvent::CommitmentTxConfirmed(funding_outpoint) |
@@ -5115,7 +5339,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                let by_id = &mut channel_state.by_id;
                                                let pending_msg_events = &mut channel_state.pending_msg_events;
                                                if let hash_map::Entry::Occupied(chan_entry) = by_id.entry(funding_outpoint.to_channel_id()) {
-                                                       let mut chan = remove_channel!(self, channel_state, chan_entry);
+                                                       let mut chan = remove_channel!(self, chan_entry);
                                                        failed_channels.push(chan.force_shutdown(false));
                                                        if let Ok(update) = self.get_channel_update_for_broadcast(&chan) {
                                                                pending_msg_events.push(events::MessageSendEvent::BroadcastChannelUpdate {
@@ -5136,7 +5360,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                        });
                                                }
                                        },
-                                       MonitorEvent::UpdateCompleted { funding_txo, monitor_update_id } => {
+                                       MonitorEvent::Completed { funding_txo, monitor_update_id } => {
                                                self.channel_monitor_updated(&funding_txo, monitor_update_id);
                                        },
                                }
@@ -5174,7 +5398,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        let mut channel_state_lock = self.channel_state.lock().unwrap();
                        let channel_state = &mut *channel_state_lock;
                        let by_id = &mut channel_state.by_id;
-                       let short_to_chan_info = &mut channel_state.short_to_chan_info;
                        let pending_msg_events = &mut channel_state.pending_msg_events;
 
                        by_id.retain(|channel_id, chan| {
@@ -5188,22 +5411,25 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                                                        ));
                                                }
                                                if let Some((commitment_update, monitor_update)) = commitment_opt {
-                                                       if let Err(e) = self.chain_monitor.update_channel(chan.get_funding_txo().unwrap(), monitor_update) {
-                                                               has_monitor_update = true;
-                                                               let (res, close_channel) = handle_monitor_err!(self, e, short_to_chan_info, chan, RAACommitmentOrder::CommitmentFirst, channel_id, COMMITMENT_UPDATE_ONLY);
-                                                               handle_errors.push((chan.get_counterparty_node_id(), res));
-                                                               if close_channel { return false; }
-                                                       } else {
-                                                               pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
-                                                                       node_id: chan.get_counterparty_node_id(),
-                                                                       updates: commitment_update,
-                                                               });
+                                                       match self.chain_monitor.update_channel(chan.get_funding_txo().unwrap(), monitor_update) {
+                                                               ChannelMonitorUpdateStatus::Completed => {
+                                                                       pending_msg_events.push(events::MessageSendEvent::UpdateHTLCs {
+                                                                               node_id: chan.get_counterparty_node_id(),
+                                                                               updates: commitment_update,
+                                                                       });
+                                                               },
+                                                               e => {
+                                                                       has_monitor_update = true;
+                                                                       let (res, close_channel) = handle_monitor_update_res!(self, e, chan, RAACommitmentOrder::CommitmentFirst, channel_id, COMMITMENT_UPDATE_ONLY);
+                                                                       handle_errors.push((chan.get_counterparty_node_id(), res));
+                                                                       if close_channel { return false; }
+                                                               },
                                                        }
                                                }
                                                true
                                        },
                                        Err(e) => {
-                                               let (close_channel, res) = convert_chan_err!(self, e, short_to_chan_info, chan, channel_id);
+                                               let (close_channel, res) = convert_chan_err!(self, e, chan, channel_id);
                                                handle_errors.push((chan.get_counterparty_node_id(), Err(res)));
                                                // ChannelClosed event is generated by handle_error for us
                                                !close_channel
@@ -5234,7 +5460,6 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
                        let mut channel_state_lock = self.channel_state.lock().unwrap();
                        let channel_state = &mut *channel_state_lock;
                        let by_id = &mut channel_state.by_id;
-                       let short_to_chan_info = &mut channel_state.short_to_chan_info;
                        let pending_msg_events = &mut channel_state.pending_msg_events;
 
                        by_id.retain(|channel_id, chan| {
@@ -5259,13 +5484,13 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
 
                                                        log_info!(self.logger, "Broadcasting {}", log_tx!(tx));
                                                        self.tx_broadcaster.broadcast_transaction(&tx);
-                                                       update_maps_on_chan_removal!(self, short_to_chan_info, chan);
+                                                       update_maps_on_chan_removal!(self, chan);
                                                        false
                                                } else { true }
                                        },
                                        Err(e) => {
                                                has_update = true;
-                                               let (close_channel, res) = convert_chan_err!(self, e, short_to_chan_info, chan, channel_id);
+                                               let (close_channel, res) = convert_chan_err!(self, e, chan, channel_id);
                                                handle_errors.push((chan.get_counterparty_node_id(), Err(res)));
                                                !close_channel
                                        }
@@ -5455,14 +5680,14 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        ///
        /// [phantom node payments]: crate::chain::keysinterface::PhantomKeysManager
        pub fn get_phantom_scid(&self) -> u64 {
-               let mut channel_state = self.channel_state.lock().unwrap();
-               let best_block = self.best_block.read().unwrap();
+               let best_block_height = self.best_block.read().unwrap().height();
+               let short_to_chan_info = self.short_to_chan_info.read().unwrap();
                loop {
-                       let scid_candidate = fake_scid::Namespace::Phantom.get_fake_scid(best_block.height(), &self.genesis_hash, &self.fake_scid_rand_bytes, &self.keys_manager);
+                       let scid_candidate = fake_scid::Namespace::Phantom.get_fake_scid(best_block_height, &self.genesis_hash, &self.fake_scid_rand_bytes, &self.keys_manager);
                        // Ensure the generated scid doesn't conflict with a real channel.
-                       match channel_state.short_to_chan_info.entry(scid_candidate) {
-                               hash_map::Entry::Occupied(_) => continue,
-                               hash_map::Entry::Vacant(_) => return scid_candidate
+                       match short_to_chan_info.get(&scid_candidate) {
+                               Some(_) => continue,
+                               None => return scid_candidate
                        }
                }
        }
@@ -5481,7 +5706,7 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        #[cfg(any(test, fuzzing, feature = "_test_utils"))]
        pub fn get_and_clear_pending_events(&self) -> Vec<events::Event> {
                let events = core::cell::RefCell::new(Vec::new());
-               let event_handler = |event: &events::Event| events.borrow_mut().push(event.clone());
+               let event_handler = |event: events::Event| events.borrow_mut().push(event);
                self.process_pending_events(&event_handler);
                events.into_inner()
        }
@@ -5495,12 +5720,45 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelMana
        pub fn clear_pending_payments(&self) {
                self.pending_outbound_payments.lock().unwrap().clear()
        }
+
+       /// Processes any events asynchronously in the order they were generated since the last call
+       /// using the given event handler.
+       ///
+       /// See the trait-level documentation of [`EventsProvider`] for requirements.
+       pub async fn process_pending_events_async<Future: core::future::Future, H: Fn(Event) -> Future>(
+               &self, handler: H
+       ) {
+               // We'll acquire our total consistency lock until the returned future completes so that
+               // we can be sure no other persists happen while processing events.
+               let _read_guard = self.total_consistency_lock.read().unwrap();
+
+               let mut result = NotifyOption::SkipPersist;
+
+               // TODO: This behavior should be documented. It's unintuitive that we query
+               // ChannelMonitors when clearing other events.
+               if self.process_pending_monitor_events() {
+                       result = NotifyOption::DoPersist;
+               }
+
+               let pending_events = mem::replace(&mut *self.pending_events.lock().unwrap(), vec![]);
+               if !pending_events.is_empty() {
+                       result = NotifyOption::DoPersist;
+               }
+
+               for event in pending_events {
+                       handler(event).await;
+               }
+
+               if result == NotifyOption::DoPersist {
+                       self.persistence_notifier.notify();
+               }
+       }
 }
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> MessageSendEventsProvider for ChannelManager<Signer, M, T, K, F, L>
-       where M::Target: chain::Watch<Signer>,
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> MessageSendEventsProvider for ChannelManager<M, T, K, F, L>
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
                                L::Target: Logger,
 {
@@ -5536,11 +5794,11 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> MessageSend
        }
 }
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> EventsProvider for ChannelManager<Signer, M, T, K, F, L>
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> EventsProvider for ChannelManager<M, T, K, F, L>
 where
-       M::Target: chain::Watch<Signer>,
+       M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
        T::Target: BroadcasterInterface,
-       K::Target: KeysInterface<Signer = Signer>,
+       K::Target: KeysInterface,
        F::Target: FeeEstimator,
        L::Target: Logger,
 {
@@ -5558,13 +5816,13 @@ where
                                result = NotifyOption::DoPersist;
                        }
 
-                       let mut pending_events = mem::replace(&mut *self.pending_events.lock().unwrap(), vec![]);
+                       let pending_events = mem::replace(&mut *self.pending_events.lock().unwrap(), vec![]);
                        if !pending_events.is_empty() {
                                result = NotifyOption::DoPersist;
                        }
 
-                       for event in pending_events.drain(..) {
-                               handler.handle_event(&event);
+                       for event in pending_events {
+                               handler.handle_event(event);
                        }
 
                        result
@@ -5572,11 +5830,11 @@ where
        }
 }
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> chain::Listen for ChannelManager<Signer, M, T, K, F, L>
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> chain::Listen for ChannelManager<M, T, K, F, L>
 where
-       M::Target: chain::Watch<Signer>,
+       M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
        T::Target: BroadcasterInterface,
-       K::Target: KeysInterface<Signer = Signer>,
+       K::Target: KeysInterface,
        F::Target: FeeEstimator,
        L::Target: Logger,
 {
@@ -5609,11 +5867,11 @@ where
        }
 }
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> chain::Confirm for ChannelManager<Signer, M, T, K, F, L>
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> chain::Confirm for ChannelManager<M, T, K, F, L>
 where
-       M::Target: chain::Watch<Signer>,
+       M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
        T::Target: BroadcasterInterface,
-       K::Target: KeysInterface<Signer = Signer>,
+       K::Target: KeysInterface,
        F::Target: FeeEstimator,
        L::Target: Logger,
 {
@@ -5671,29 +5929,14 @@ where
                payment_secrets.retain(|_, inbound_payment| {
                        inbound_payment.expiry_time > header.time as u64
                });
-
-               let mut outbounds = self.pending_outbound_payments.lock().unwrap();
-               let mut pending_events = self.pending_events.lock().unwrap();
-               outbounds.retain(|payment_id, payment| {
-                       if payment.remaining_parts() != 0 { return true }
-                       if let PendingOutboundPayment::Retryable { starting_block_height, payment_hash, .. } = payment {
-                               if *starting_block_height + PAYMENT_EXPIRY_BLOCKS <= height {
-                                       log_info!(self.logger, "Timing out payment with id {} and hash {}", log_bytes!(payment_id.0), log_bytes!(payment_hash.0));
-                                       pending_events.push(events::Event::PaymentFailed {
-                                               payment_id: *payment_id, payment_hash: *payment_hash,
-                                       });
-                                       false
-                               } else { true }
-                       } else { true }
-               });
        }
 
-       fn get_relevant_txids(&self) -> Vec<Txid> {
+       fn get_relevant_txids(&self) -> Vec<(Txid, Option<BlockHash>)> {
                let channel_state = self.channel_state.lock().unwrap();
-               let mut res = Vec::with_capacity(channel_state.short_to_chan_info.len());
+               let mut res = Vec::with_capacity(channel_state.by_id.len());
                for chan in channel_state.by_id.values() {
-                       if let Some(funding_txo) = chan.get_funding_txo() {
-                               res.push(funding_txo.txid);
+                       if let (Some(funding_txo), block_hash) = (chan.get_funding_txo(), chan.get_funding_tx_confirmed_in()) {
+                               res.push((funding_txo.txid, block_hash));
                        }
                }
                res
@@ -5711,18 +5954,18 @@ where
        }
 }
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelManager<Signer, M, T, K, F, L>
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> ChannelManager<M, T, K, F, L>
 where
-       M::Target: chain::Watch<Signer>,
+       M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
        T::Target: BroadcasterInterface,
-       K::Target: KeysInterface<Signer = Signer>,
+       K::Target: KeysInterface,
        F::Target: FeeEstimator,
        L::Target: Logger,
 {
        /// Calls a function which handles an on-chain event (blocks dis/connected, transactions
        /// un/confirmed, etc) on each channel, handling any resulting errors or messages generated by
        /// the function.
-       fn do_chain_event<FN: Fn(&mut Channel<Signer>) -> Result<(Option<msgs::ChannelReady>, Vec<(HTLCSource, PaymentHash)>, Option<msgs::AnnouncementSignatures>), ClosureReason>>
+       fn do_chain_event<FN: Fn(&mut Channel<<K::Target as KeysInterface>::Signer>) -> Result<(Option<msgs::ChannelReady>, Vec<(HTLCSource, PaymentHash)>, Option<msgs::AnnouncementSignatures>), ClosureReason>>
                        (&self, height_opt: Option<u32>, f: FN) {
                // Note that we MUST NOT end up calling methods on self.chain_monitor here - we're called
                // during initialization prior to the chain_monitor being fully configured in some cases.
@@ -5733,7 +5976,6 @@ where
                {
                        let mut channel_lock = self.channel_state.lock().unwrap();
                        let channel_state = &mut *channel_lock;
-                       let short_to_chan_info = &mut channel_state.short_to_chan_info;
                        let pending_msg_events = &mut channel_state.pending_msg_events;
                        channel_state.by_id.retain(|_, channel| {
                                let res = f(channel);
@@ -5745,7 +5987,7 @@ where
                                                }, HTLCDestination::NextHopChannel { node_id: Some(channel.get_counterparty_node_id()), channel_id: channel.channel_id() }));
                                        }
                                        if let Some(channel_ready) = channel_ready_opt {
-                                               send_channel_ready!(short_to_chan_info, pending_msg_events, channel, channel_ready);
+                                               send_channel_ready!(self, pending_msg_events, channel, channel_ready);
                                                if channel.is_usable() {
                                                        log_trace!(self.logger, "Sending channel_ready with private initial channel_update for our counterparty on channel {}", log_bytes!(channel.channel_id()));
                                                        if let Ok(msg) = self.get_channel_update_for_unicast(channel) {
@@ -5758,6 +6000,9 @@ where
                                                        log_trace!(self.logger, "Sending channel_ready WITHOUT channel_update for {}", log_bytes!(channel.channel_id()));
                                                }
                                        }
+
+                                       emit_channel_ready_event!(self, channel);
+
                                        if let Some(announcement_sigs) = announcement_sigs {
                                                log_trace!(self.logger, "Sending announcement_signatures for channel {}", log_bytes!(channel.channel_id()));
                                                pending_msg_events.push(events::MessageSendEvent::SendAnnouncementSignatures {
@@ -5783,6 +6028,7 @@ where
                                                        // enforce option_scid_alias then), and if the funding tx is ever
                                                        // un-confirmed we force-close the channel, ensuring short_to_chan_info
                                                        // is always consistent.
+                                                       let mut short_to_chan_info = self.short_to_chan_info.write().unwrap();
                                                        let scid_insert = short_to_chan_info.insert(real_scid, (channel.get_counterparty_node_id(), channel.channel_id()));
                                                        assert!(scid_insert.is_none() || scid_insert.unwrap() == (channel.get_counterparty_node_id(), channel.channel_id()),
                                                                "SCIDs should never collide - ensure you weren't behind by a full {} blocks when creating channels",
@@ -5790,7 +6036,7 @@ where
                                                }
                                        }
                                } else if let Err(reason) = res {
-                                       update_maps_on_chan_removal!(self, short_to_chan_info, channel);
+                                       update_maps_on_chan_removal!(self, channel);
                                        // It looks like our counterparty went on-chain or funding transaction was
                                        // reorged out of the main chain. Close the channel.
                                        failed_channels.push(channel.force_shutdown(true));
@@ -5839,7 +6085,7 @@ where
                self.handle_init_event_channel_failures(failed_channels);
 
                for (source, payment_hash, reason, destination) in timed_out_htlcs.drain(..) {
-                       self.fail_htlc_backwards_internal(self.channel_state.lock().unwrap(), source, &payment_hash, reason, destination);
+                       self.fail_htlc_backwards_internal(source, &payment_hash, reason, destination);
                }
        }
 
@@ -5880,11 +6126,11 @@ where
        }
 }
 
-impl<Signer: Sign, M: Deref , T: Deref , K: Deref , F: Deref , L: Deref >
-       ChannelMessageHandler for ChannelManager<Signer, M, T, K, F, L>
-       where M::Target: chain::Watch<Signer>,
+impl<M: Deref , T: Deref , K: Deref , F: Deref , L: Deref >
+       ChannelMessageHandler for ChannelManager<M, T, K, F, L>
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
         L::Target: Logger,
 {
@@ -5986,14 +6232,13 @@ impl<Signer: Sign, M: Deref , T: Deref , K: Deref , F: Deref , L: Deref >
                        let mut channel_state_lock = self.channel_state.lock().unwrap();
                        let channel_state = &mut *channel_state_lock;
                        let pending_msg_events = &mut channel_state.pending_msg_events;
-                       let short_to_chan_info = &mut channel_state.short_to_chan_info;
                        log_debug!(self.logger, "Marking channels with {} disconnected and generating channel_updates. We believe we {} make future connections to this peer.",
                                log_pubkey!(counterparty_node_id), if no_connection_possible { "cannot" } else { "can" });
                        channel_state.by_id.retain(|_, chan| {
                                if chan.get_counterparty_node_id() == *counterparty_node_id {
                                        chan.remove_uncommitted_htlcs_and_mark_paused(&self.logger);
                                        if chan.is_shutdown() {
-                                               update_maps_on_chan_removal!(self, short_to_chan_info, chan);
+                                               update_maps_on_chan_removal!(self, chan);
                                                self.issue_channel_close_events(chan, ClosureReason::DisconnectedPeer);
                                                return false;
                                        } else {
@@ -6252,8 +6497,9 @@ impl_writeable_tlv_based!(PendingHTLCInfo, {
        (0, routing, required),
        (2, incoming_shared_secret, required),
        (4, payment_hash, required),
-       (6, amt_to_forward, required),
-       (8, outgoing_cltv_value, required)
+       (6, outgoing_amt_msat, required),
+       (8, outgoing_cltv_value, required),
+       (9, incoming_amt_msat, option),
 });
 
 
@@ -6357,7 +6603,7 @@ impl Writeable for ClaimableHTLC {
 
 impl Readable for ClaimableHTLC {
        fn read<R: Read>(reader: &mut R) -> Result<Self, DecodeError> {
-               let mut prev_hop = ::util::ser::OptionDeserWrapper(None);
+               let mut prev_hop = crate::util::ser::OptionDeserWrapper(None);
                let mut value = 0;
                let mut payment_data: Option<msgs::FinalOnionHopData> = None;
                let mut cltv_expiry = 0;
@@ -6407,7 +6653,7 @@ impl Readable for HTLCSource {
                let id: u8 = Readable::read(reader)?;
                match id {
                        0 => {
-                               let mut session_priv: ::util::ser::OptionDeserWrapper<SecretKey> = ::util::ser::OptionDeserWrapper(None);
+                               let mut session_priv: crate::util::ser::OptionDeserWrapper<SecretKey> = crate::util::ser::OptionDeserWrapper(None);
                                let mut first_hop_htlc_msat: u64 = 0;
                                let mut path = Some(Vec::new());
                                let mut payment_id = None;
@@ -6428,7 +6674,7 @@ impl Readable for HTLCSource {
                                }
                                Ok(HTLCSource::OutboundRoute {
                                        session_priv: session_priv.0.unwrap(),
-                                       first_hop_htlc_msat: first_hop_htlc_msat,
+                                       first_hop_htlc_msat,
                                        path: path.unwrap(),
                                        payment_id: payment_id.unwrap(),
                                        payment_secret,
@@ -6442,7 +6688,7 @@ impl Readable for HTLCSource {
 }
 
 impl Writeable for HTLCSource {
-       fn write<W: Writer>(&self, writer: &mut W) -> Result<(), ::io::Error> {
+       fn write<W: Writer>(&self, writer: &mut W) -> Result<(), crate::io::Error> {
                match self {
                        HTLCSource::OutboundRoute { ref session_priv, ref first_hop_htlc_msat, ref path, payment_id, payment_secret, payment_params } => {
                                0u8.write(writer)?;
@@ -6452,7 +6698,7 @@ impl Writeable for HTLCSource {
                                        (1, payment_id_opt, option),
                                        (2, first_hop_htlc_msat, required),
                                        (3, payment_secret, option),
-                                       (4, path, vec_type),
+                                       (4, *path, vec_type),
                                        (5, payment_params, option),
                                 });
                        }
@@ -6475,18 +6721,20 @@ impl_writeable_tlv_based_enum!(HTLCFailReason,
        },
 ;);
 
+impl_writeable_tlv_based!(PendingAddHTLCInfo, {
+       (0, forward_info, required),
+       (2, prev_short_channel_id, required),
+       (4, prev_htlc_id, required),
+       (6, prev_funding_outpoint, required),
+});
+
 impl_writeable_tlv_based_enum!(HTLCForwardInfo,
-       (0, AddHTLC) => {
-               (0, forward_info, required),
-               (2, prev_short_channel_id, required),
-               (4, prev_htlc_id, required),
-               (6, prev_funding_outpoint, required),
-       },
        (1, FailHTLC) => {
                (0, htlc_id, required),
                (2, err_packet, required),
-       },
-;);
+       };
+       (0, AddHTLC)
+);
 
 impl_writeable_tlv_based!(PendingInboundPayment, {
        (0, payment_secret, required),
@@ -6503,6 +6751,7 @@ impl_writeable_tlv_based_enum_upgradable!(PendingOutboundPayment,
        (1, Fulfilled) => {
                (0, session_privs, required),
                (1, payment_hash, option),
+               (3, timer_ticks_without_htlcs, (default_value, 0)),
        },
        (2, Retryable) => {
                (0, session_privs, required),
@@ -6519,10 +6768,10 @@ impl_writeable_tlv_based_enum_upgradable!(PendingOutboundPayment,
        },
 );
 
-impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> Writeable for ChannelManager<Signer, M, T, K, F, L>
-       where M::Target: chain::Watch<Signer>,
+impl<M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> Writeable for ChannelManager<M, T, K, F, L>
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
         L::Target: Logger,
 {
@@ -6538,29 +6787,37 @@ impl<Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref> Writeable f
                        best_block.block_hash().write(writer)?;
                }
 
-               let channel_state = self.channel_state.lock().unwrap();
-               let mut unfunded_channels = 0;
-               for (_, channel) in channel_state.by_id.iter() {
-                       if !channel.is_funding_initiated() {
-                               unfunded_channels += 1;
+               {
+                       // Take `channel_state` lock temporarily to avoid creating a lock order that requires
+                       // that the `forward_htlcs` lock is taken after `channel_state`
+                       let channel_state = self.channel_state.lock().unwrap();
+                       let mut unfunded_channels = 0;
+                       for (_, channel) in channel_state.by_id.iter() {
+                               if !channel.is_funding_initiated() {
+                                       unfunded_channels += 1;
+                               }
                        }
-               }
-               ((channel_state.by_id.len() - unfunded_channels) as u64).write(writer)?;
-               for (_, channel) in channel_state.by_id.iter() {
-                       if channel.is_funding_initiated() {
-                               channel.write(writer)?;
+                       ((channel_state.by_id.len() - unfunded_channels) as u64).write(writer)?;
+                       for (_, channel) in channel_state.by_id.iter() {
+                               if channel.is_funding_initiated() {
+                                       channel.write(writer)?;
+                               }
                        }
                }
 
-               (channel_state.forward_htlcs.len() as u64).write(writer)?;
-               for (short_channel_id, pending_forwards) in channel_state.forward_htlcs.iter() {
-                       short_channel_id.write(writer)?;
-                       (pending_forwards.len() as u64).write(writer)?;
-                       for forward in pending_forwards {
-                               forward.write(writer)?;
+               {
+                       let forward_htlcs = self.forward_htlcs.lock().unwrap();
+                       (forward_htlcs.len() as u64).write(writer)?;
+                       for (short_channel_id, pending_forwards) in forward_htlcs.iter() {
+                               short_channel_id.write(writer)?;
+                               (pending_forwards.len() as u64).write(writer)?;
+                               for forward in pending_forwards {
+                                       forward.write(writer)?;
+                               }
                        }
                }
 
+               let channel_state = self.channel_state.lock().unwrap();
                let mut htlc_purposes: Vec<&events::PaymentPurpose> = Vec::new();
                (channel_state.claimable_htlcs.len() as u64).write(writer)?;
                for (payment_hash, (purpose, previous_hops)) in channel_state.claimable_htlcs.iter() {
@@ -6760,29 +7017,29 @@ impl<'a, Signer: 'a + Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
 
 // Implement ReadableArgs for an Arc'd ChannelManager to make it a bit easier to work with the
 // SipmleArcChannelManager type:
-impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
-       ReadableArgs<ChannelManagerReadArgs<'a, Signer, M, T, K, F, L>> for (BlockHash, Arc<ChannelManager<Signer, M, T, K, F, L>>)
-       where M::Target: chain::Watch<Signer>,
+impl<'a, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
+       ReadableArgs<ChannelManagerReadArgs<'a, <K::Target as KeysInterface>::Signer, M, T, K, F, L>> for (BlockHash, Arc<ChannelManager<M, T, K, F, L>>)
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
         L::Target: Logger,
 {
-       fn read<R: io::Read>(reader: &mut R, args: ChannelManagerReadArgs<'a, Signer, M, T, K, F, L>) -> Result<Self, DecodeError> {
-               let (blockhash, chan_manager) = <(BlockHash, ChannelManager<Signer, M, T, K, F, L>)>::read(reader, args)?;
+       fn read<R: io::Read>(reader: &mut R, args: ChannelManagerReadArgs<'a, <K::Target as KeysInterface>::Signer, M, T, K, F, L>) -> Result<Self, DecodeError> {
+               let (blockhash, chan_manager) = <(BlockHash, ChannelManager<M, T, K, F, L>)>::read(reader, args)?;
                Ok((blockhash, Arc::new(chan_manager)))
        }
 }
 
-impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
-       ReadableArgs<ChannelManagerReadArgs<'a, Signer, M, T, K, F, L>> for (BlockHash, ChannelManager<Signer, M, T, K, F, L>)
-       where M::Target: chain::Watch<Signer>,
+impl<'a, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
+       ReadableArgs<ChannelManagerReadArgs<'a, <K::Target as KeysInterface>::Signer, M, T, K, F, L>> for (BlockHash, ChannelManager<M, T, K, F, L>)
+       where M::Target: chain::Watch<<K::Target as KeysInterface>::Signer>,
         T::Target: BroadcasterInterface,
-        K::Target: KeysInterface<Signer = Signer>,
+        K::Target: KeysInterface,
         F::Target: FeeEstimator,
         L::Target: Logger,
 {
-       fn read<R: io::Read>(reader: &mut R, mut args: ChannelManagerReadArgs<'a, Signer, M, T, K, F, L>) -> Result<Self, DecodeError> {
+       fn read<R: io::Read>(reader: &mut R, mut args: ChannelManagerReadArgs<'a, <K::Target as KeysInterface>::Signer, M, T, K, F, L>) -> Result<Self, DecodeError> {
                let _ver = read_ver_prefix!(reader, SERIALIZATION_VERSION);
 
                let genesis_hash: BlockHash = Readable::read(reader)?;
@@ -6798,7 +7055,7 @@ impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
                let mut short_to_chan_info = HashMap::with_capacity(cmp::min(channel_count as usize, 128));
                let mut channel_closures = Vec::new();
                for _ in 0..channel_count {
-                       let mut channel: Channel<Signer> = Channel::read(reader, (&args.keys_manager, best_block_height))?;
+                       let mut channel: Channel<<K::Target as KeysInterface>::Signer> = Channel::read(reader, (&args.keys_manager, best_block_height))?;
                        let funding_txo = channel.get_funding_txo().ok_or(DecodeError::InvalidValue)?;
                        funding_txo_set.insert(funding_txo.clone());
                        if let Some(ref mut monitor) = args.channel_monitors.get_mut(&funding_txo) {
@@ -6842,6 +7099,16 @@ impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
                                        }
                                        by_id.insert(channel.channel_id(), channel);
                                }
+                       } else if channel.is_awaiting_initial_mon_persist() {
+                               // If we were persisted and shut down while the initial ChannelMonitor persistence
+                               // was in-progress, we never broadcasted the funding transaction and can still
+                               // safely discard the channel.
+                               let _ = channel.force_shutdown(false);
+                               channel_closures.push(events::Event::ChannelClosed {
+                                       channel_id: channel.channel_id(),
+                                       user_channel_id: channel.get_user_id(),
+                                       reason: ClosureReason::DisconnectedPeer,
+                               });
                        } else {
                                log_error!(args.logger, "Missing ChannelMonitor for channel {} needed by ChannelManager.", log_bytes!(channel.channel_id()));
                                log_error!(args.logger, " The chain::Watch API *requires* that monitors are persisted durably before returning,");
@@ -7164,8 +7431,6 @@ impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
 
                        channel_state: Mutex::new(ChannelHolder {
                                by_id,
-                               short_to_chan_info,
-                               forward_htlcs,
                                claimable_htlcs,
                                pending_msg_events: Vec::new(),
                        }),
@@ -7173,8 +7438,10 @@ impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
                        pending_inbound_payments: Mutex::new(pending_inbound_payments),
                        pending_outbound_payments: Mutex::new(pending_outbound_payments.unwrap()),
 
+                       forward_htlcs: Mutex::new(forward_htlcs),
                        outbound_scid_aliases: Mutex::new(outbound_scid_aliases),
                        id_to_peer: Mutex::new(id_to_peer),
+                       short_to_chan_info: FairRwLock::new(short_to_chan_info),
                        fake_scid_rand_bytes: fake_scid_rand_bytes.unwrap(),
 
                        probing_cookie_secret: probing_cookie_secret.unwrap(),
@@ -7200,7 +7467,7 @@ impl<'a, Signer: Sign, M: Deref, T: Deref, K: Deref, F: Deref, L: Deref>
                for htlc_source in failed_htlcs.drain(..) {
                        let (source, payment_hash, counterparty_node_id, channel_id) = htlc_source;
                        let receiver = HTLCDestination::NextHopChannel { node_id: Some(counterparty_node_id), channel_id };
-                       channel_manager.fail_htlc_backwards_internal(channel_manager.channel_state.lock().unwrap(), source, &payment_hash, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
+                       channel_manager.fail_htlc_backwards_internal(source, &payment_hash, HTLCFailReason::Reason { failure_code: 0x4000 | 8, data: Vec::new() }, receiver);
                }
 
                //TODO: Broadcast channel update for closed channels, but only after we've made a
@@ -7216,16 +7483,16 @@ mod tests {
        use bitcoin::hashes::sha256::Hash as Sha256;
        use core::time::Duration;
        use core::sync::atomic::Ordering;
-       use ln::{PaymentPreimage, PaymentHash, PaymentSecret};
-       use ln::channelmanager::{self, inbound_payment, PaymentId, PaymentSendFailure};
-       use ln::functional_test_utils::*;
-       use ln::msgs;
-       use ln::msgs::ChannelMessageHandler;
-       use routing::router::{PaymentParameters, RouteParameters, find_route};
-       use util::errors::APIError;
-       use util::events::{Event, HTLCDestination, MessageSendEvent, MessageSendEventsProvider, ClosureReason};
-       use util::test_utils;
-       use chain::keysinterface::KeysInterface;
+       use crate::ln::{PaymentPreimage, PaymentHash, PaymentSecret};
+       use crate::ln::channelmanager::{self, inbound_payment, PaymentId, PaymentSendFailure};
+       use crate::ln::functional_test_utils::*;
+       use crate::ln::msgs;
+       use crate::ln::msgs::ChannelMessageHandler;
+       use crate::routing::router::{PaymentParameters, RouteParameters, find_route};
+       use crate::util::errors::APIError;
+       use crate::util::events::{Event, HTLCDestination, MessageSendEvent, MessageSendEventsProvider, ClosureReason};
+       use crate::util::test_utils;
+       use crate::chain::keysinterface::KeysInterface;
 
        #[test]
        fn test_notify_limits() {
@@ -7317,18 +7584,22 @@ mod tests {
 
                // First, send a partial MPP payment.
                let (route, our_payment_hash, payment_preimage, payment_secret) = get_route_and_payment_hash!(&nodes[0], nodes[1], 100_000);
+               let mut mpp_route = route.clone();
+               mpp_route.paths.push(mpp_route.paths[0].clone());
+
                let payment_id = PaymentId([42; 32]);
                // Use the utility function send_payment_along_path to send the payment with MPP data which
                // indicates there are more HTLCs coming.
                let cur_height = CHAN_CONFIRM_DEPTH + 1; // route_payment calls send_payment, which adds 1 to the current height. So we do the same here to match.
-               nodes[0].node.send_payment_along_path(&route.paths[0], &route.payment_params, &our_payment_hash, &Some(payment_secret), 200_000, cur_height, payment_id, &None).unwrap();
+               let session_privs = nodes[0].node.add_new_pending_payment(our_payment_hash, Some(payment_secret), payment_id, &mpp_route).unwrap();
+               nodes[0].node.send_payment_along_path(&mpp_route.paths[0], &route.payment_params, &our_payment_hash, &Some(payment_secret), 200_000, cur_height, payment_id, &None, session_privs[0]).unwrap();
                check_added_monitors!(nodes[0], 1);
                let mut events = nodes[0].node.get_and_clear_pending_msg_events();
                assert_eq!(events.len(), 1);
                pass_along_path(&nodes[0], &[&nodes[1]], 200_000, our_payment_hash, Some(payment_secret), events.drain(..).next().unwrap(), false, None);
 
                // Next, send a keysend payment with the same payment_hash and make sure it fails.
-               nodes[0].node.send_spontaneous_payment(&route, Some(payment_preimage)).unwrap();
+               nodes[0].node.send_spontaneous_payment(&route, Some(payment_preimage), PaymentId(payment_preimage.0)).unwrap();
                check_added_monitors!(nodes[0], 1);
                let mut events = nodes[0].node.get_and_clear_pending_msg_events();
                assert_eq!(events.len(), 1);
@@ -7351,7 +7622,7 @@ mod tests {
                expect_payment_failed!(nodes[0], our_payment_hash, true);
 
                // Send the second half of the original MPP payment.
-               nodes[0].node.send_payment_along_path(&route.paths[0], &route.payment_params, &our_payment_hash, &Some(payment_secret), 200_000, cur_height, payment_id, &None).unwrap();
+               nodes[0].node.send_payment_along_path(&mpp_route.paths[1], &route.payment_params, &our_payment_hash, &Some(payment_secret), 200_000, cur_height, payment_id, &None, session_privs[1]).unwrap();
                check_added_monitors!(nodes[0], 1);
                let mut events = nodes[0].node.get_and_clear_pending_msg_events();
                assert_eq!(events.len(), 1);
@@ -7449,7 +7720,7 @@ mod tests {
                        &nodes[0].node.get_our_node_id(), &route_params, &nodes[0].network_graph,
                        None, nodes[0].logger, &scorer, &random_seed_bytes
                ).unwrap();
-               nodes[0].node.send_spontaneous_payment(&route, Some(payment_preimage)).unwrap();
+               nodes[0].node.send_spontaneous_payment(&route, Some(payment_preimage), PaymentId(payment_preimage.0)).unwrap();
                check_added_monitors!(nodes[0], 1);
                let mut events = nodes[0].node.get_and_clear_pending_msg_events();
                assert_eq!(events.len(), 1);
@@ -7482,7 +7753,7 @@ mod tests {
                        &nodes[0].node.get_our_node_id(), &route_params, &nodes[0].network_graph,
                        None, nodes[0].logger, &scorer, &random_seed_bytes
                ).unwrap();
-               let (payment_hash, _) = nodes[0].node.send_spontaneous_payment(&route, Some(payment_preimage)).unwrap();
+               let payment_hash = nodes[0].node.send_spontaneous_payment(&route, Some(payment_preimage), PaymentId(payment_preimage.0)).unwrap();
                check_added_monitors!(nodes[0], 1);
                let mut events = nodes[0].node.get_and_clear_pending_msg_events();
                assert_eq!(events.len(), 1);
@@ -7492,7 +7763,7 @@ mod tests {
 
                // Next, attempt a regular payment and make sure it fails.
                let payment_secret = PaymentSecret([43; 32]);
-               nodes[0].node.send_payment(&route, payment_hash, &Some(payment_secret)).unwrap();
+               nodes[0].node.send_payment(&route, payment_hash, &Some(payment_secret), PaymentId(payment_hash.0)).unwrap();
                check_added_monitors!(nodes[0], 1);
                let mut events = nodes[0].node.get_and_clear_pending_msg_events();
                assert_eq!(events.len(), 1);
@@ -7535,7 +7806,7 @@ mod tests {
                let _chan = create_chan_between_nodes(&nodes[0], &nodes[1], channelmanager::provided_init_features(), channelmanager::provided_init_features());
                let route_params = RouteParameters {
                        payment_params: PaymentParameters::for_keysend(payee_pubkey),
-                       final_value_msat: 10000,
+                       final_value_msat: 10_000,
                        final_cltv_expiry_delta: 40,
                };
                let network_graph = nodes[0].network_graph;
@@ -7549,7 +7820,8 @@ mod tests {
 
                let test_preimage = PaymentPreimage([42; 32]);
                let mismatch_payment_hash = PaymentHash([43; 32]);
-               let _ = nodes[0].node.send_payment_internal(&route, mismatch_payment_hash, &None, Some(test_preimage), None, None).unwrap();
+               let session_privs = nodes[0].node.add_new_pending_payment(mismatch_payment_hash, None, PaymentId(mismatch_payment_hash.0), &route).unwrap();
+               nodes[0].node.send_payment_internal(&route, mismatch_payment_hash, &None, Some(test_preimage), PaymentId(mismatch_payment_hash.0), None, session_privs).unwrap();
                check_added_monitors!(nodes[0], 1);
 
                let updates = get_htlc_update_msgs!(nodes[0], nodes[1].node.get_our_node_id());
@@ -7579,7 +7851,7 @@ mod tests {
                let _chan = create_chan_between_nodes(&nodes[0], &nodes[1], channelmanager::provided_init_features(), channelmanager::provided_init_features());
                let route_params = RouteParameters {
                        payment_params: PaymentParameters::for_keysend(payee_pubkey),
-                       final_value_msat: 10000,
+                       final_value_msat: 10_000,
                        final_cltv_expiry_delta: 40,
                };
                let network_graph = nodes[0].network_graph;
@@ -7594,7 +7866,8 @@ mod tests {
                let test_preimage = PaymentPreimage([42; 32]);
                let test_secret = PaymentSecret([43; 32]);
                let payment_hash = PaymentHash(Sha256::hash(&test_preimage.0).into_inner());
-               let _ = nodes[0].node.send_payment_internal(&route, payment_hash, &Some(test_secret), Some(test_preimage), None, None).unwrap();
+               let session_privs = nodes[0].node.add_new_pending_payment(payment_hash, Some(test_secret), PaymentId(payment_hash.0), &route).unwrap();
+               nodes[0].node.send_payment_internal(&route, payment_hash, &Some(test_secret), Some(test_preimage), PaymentId(payment_hash.0), None, session_privs).unwrap();
                check_added_monitors!(nodes[0], 1);
 
                let updates = get_htlc_update_msgs!(nodes[0], nodes[1].node.get_our_node_id());
@@ -7631,7 +7904,7 @@ mod tests {
                route.paths[1][0].short_channel_id = chan_2_id;
                route.paths[1][1].short_channel_id = chan_4_id;
 
-               match nodes[0].node.send_payment(&route, payment_hash, &None).unwrap_err() {
+               match nodes[0].node.send_payment(&route, payment_hash, &None, PaymentId(payment_hash.0)).unwrap_err() {
                        PaymentSendFailure::ParameterError(APIError::APIMisuseError { ref err }) => {
                                assert!(regex::Regex::new(r"Payment secret is required for multi-path payments").unwrap().is_match(err))                        },
                        _ => panic!("unexpected error")
@@ -7783,34 +8056,33 @@ mod tests {
 
 #[cfg(all(any(test, feature = "_test_utils"), feature = "_bench_unstable"))]
 pub mod bench {
-       use chain::Listen;
-       use chain::chainmonitor::{ChainMonitor, Persist};
-       use chain::keysinterface::{KeysManager, KeysInterface, InMemorySigner};
-       use ln::channelmanager::{self, BestBlock, ChainParameters, ChannelManager, PaymentHash, PaymentPreimage};
-       use ln::features::{InitFeatures, InvoiceFeatures};
-       use ln::functional_test_utils::*;
-       use ln::msgs::{ChannelMessageHandler, Init};
-       use routing::gossip::NetworkGraph;
-       use routing::router::{PaymentParameters, get_route};
-       use util::test_utils;
-       use util::config::UserConfig;
-       use util::events::{Event, MessageSendEvent, MessageSendEventsProvider};
+       use crate::chain::Listen;
+       use crate::chain::chainmonitor::{ChainMonitor, Persist};
+       use crate::chain::keysinterface::{KeysManager, KeysInterface, InMemorySigner};
+       use crate::ln::channelmanager::{self, BestBlock, ChainParameters, ChannelManager, PaymentHash, PaymentPreimage, PaymentId};
+       use crate::ln::functional_test_utils::*;
+       use crate::ln::msgs::{ChannelMessageHandler, Init};
+       use crate::routing::gossip::NetworkGraph;
+       use crate::routing::router::{PaymentParameters, get_route};
+       use crate::util::test_utils;
+       use crate::util::config::UserConfig;
+       use crate::util::events::{Event, MessageSendEvent, MessageSendEventsProvider};
 
        use bitcoin::hashes::Hash;
        use bitcoin::hashes::sha256::Hash as Sha256;
        use bitcoin::{Block, BlockHeader, PackedLockTime, Transaction, TxMerkleNode, TxOut};
 
-       use sync::{Arc, Mutex};
+       use crate::sync::{Arc, Mutex};
 
        use test::Bencher;
 
        struct NodeHolder<'a, P: Persist<InMemorySigner>> {
-               node: &'a ChannelManager<InMemorySigner,
+               node: &'a ChannelManager<
                        &'a ChainMonitor<InMemorySigner, &'a test_utils::TestChainSource,
                                &'a test_utils::TestBroadcaster, &'a test_utils::TestFeeEstimator,
                                &'a test_utils::TestLogger, &'a P>,
                        &'a test_utils::TestBroadcaster, &'a KeysManager,
-                       &'a test_utils::TestFeeEstimator, &'a test_utils::TestLogger>
+                       &'a test_utils::TestFeeEstimator, &'a test_utils::TestLogger>,
        }
 
        #[cfg(test)]
@@ -7893,6 +8165,24 @@ pub mod bench {
                        _ => panic!(),
                }
 
+               let events_a = node_a.get_and_clear_pending_events();
+               assert_eq!(events_a.len(), 1);
+               match events_a[0] {
+                       Event::ChannelReady{ ref counterparty_node_id, .. } => {
+                               assert_eq!(*counterparty_node_id, node_b.get_our_node_id());
+                       },
+                       _ => panic!("Unexpected event"),
+               }
+
+               let events_b = node_b.get_and_clear_pending_events();
+               assert_eq!(events_b.len(), 1);
+               match events_b[0] {
+                       Event::ChannelReady{ ref counterparty_node_id, .. } => {
+                               assert_eq!(*counterparty_node_id, node_a.get_our_node_id());
+                       },
+                       _ => panic!("Unexpected event"),
+               }
+
                let dummy_graph = NetworkGraph::new(genesis_hash, &logger_a);
 
                let mut payment_count: u64 = 0;
@@ -7914,7 +8204,7 @@ pub mod bench {
                                let payment_hash = PaymentHash(Sha256::hash(&payment_preimage.0[..]).into_inner());
                                let payment_secret = $node_b.create_inbound_payment_for_hash(payment_hash, None, 7200).unwrap();
 
-                               $node_a.send_payment(&route, payment_hash, &Some(payment_secret)).unwrap();
+                               $node_a.send_payment(&route, payment_hash, &Some(payment_secret), PaymentId(payment_hash.0)).unwrap();
                                let payment_event = SendEvent::from_event($node_a.get_and_clear_pending_msg_events().pop().unwrap());
                                $node_b.handle_update_add_htlc(&$node_a.get_our_node_id(), &payment_event.msgs[0]);
                                $node_b.handle_commitment_signed(&$node_a.get_our_node_id(), &payment_event.commitment_msg);