Skip to content

Commit

Permalink
Merge pull request #1812 from valentinewallace/2022-10-chanman-router…
Browse files Browse the repository at this point in the history
…-param

Parameterize `ChannelManager` by a `Router`
  • Loading branch information
TheBlueMatt authored Jan 5, 2023
2 parents e1208bf + 19516c0 commit b79ff71
Show file tree
Hide file tree
Showing 20 changed files with 393 additions and 222 deletions.
30 changes: 25 additions & 5 deletions fuzz/src/chanmon_consistency.rs
Original file line number Diff line number Diff line change
Expand Up @@ -38,9 +38,9 @@ use lightning::chain::transaction::OutPoint;
use lightning::chain::chaininterface::{BroadcasterInterface, ConfirmationTarget, FeeEstimator};
use lightning::chain::keysinterface::{KeyMaterial, KeysInterface, InMemorySigner, Recipient, EntropySource, NodeSigner, SignerProvider};
use lightning::ln::{PaymentHash, PaymentPreimage, PaymentSecret};
use lightning::ln::channelmanager::{self, ChainParameters, ChannelManager, PaymentSendFailure, ChannelManagerReadArgs, PaymentId};
use lightning::ln::channelmanager::{self, ChainParameters, ChannelDetails, ChannelManager, PaymentSendFailure, ChannelManagerReadArgs, PaymentId};
use lightning::ln::channel::FEE_SPIKE_BUFFER_FEE_INCREASE_MULTIPLE;
use lightning::ln::msgs::{CommitmentUpdate, ChannelMessageHandler, DecodeError, UpdateAddHTLC, Init};
use lightning::ln::msgs::{self, CommitmentUpdate, ChannelMessageHandler, DecodeError, UpdateAddHTLC, Init};
use lightning::ln::script::ShutdownScript;
use lightning::util::enforcing_trait_impls::{EnforcingSigner, EnforcementState};
use lightning::util::errors::APIError;
Expand All @@ -49,7 +49,7 @@ use lightning::util::logger::Logger;
use lightning::util::config::UserConfig;
use lightning::util::events::MessageSendEventsProvider;
use lightning::util::ser::{Readable, ReadableArgs, Writeable, Writer};
use lightning::routing::router::{Route, RouteHop};
use lightning::routing::router::{InFlightHtlcs, Route, RouteHop, RouteParameters, Router};

use crate::utils::test_logger::{self, Output};
use crate::utils::test_persister::TestPersister;
Expand Down Expand Up @@ -85,6 +85,24 @@ impl FeeEstimator for FuzzEstimator {
}
}

struct FuzzRouter {}

impl Router for FuzzRouter {
fn find_route(
&self, _payer: &PublicKey, _params: &RouteParameters, _first_hops: Option<&[&ChannelDetails]>,
_inflight_htlcs: &InFlightHtlcs
) -> Result<Route, msgs::LightningError> {
Err(msgs::LightningError {
err: String::from("Not implemented"),
action: msgs::ErrorAction::IgnoreError
})
}
fn notify_payment_path_failed(&self, _path: &[&RouteHop], _short_channel_id: u64) {}
fn notify_payment_path_successful(&self, _path: &[&RouteHop]) {}
fn notify_payment_probe_successful(&self, _path: &[&RouteHop]) {}
fn notify_payment_probe_failed(&self, _path: &[&RouteHop], _short_channel_id: u64) {}
}

pub struct TestBroadcaster {}
impl BroadcasterInterface for TestBroadcaster {
fn broadcast_transaction(&self, _tx: &Transaction) { }
Expand Down Expand Up @@ -310,7 +328,7 @@ fn check_payment_err(send_err: PaymentSendFailure) {
}
}

type ChanMan = ChannelManager<Arc<TestChainMonitor>, Arc<TestBroadcaster>, Arc<KeyProvider>, Arc<FuzzEstimator>, Arc<dyn Logger>>;
type ChanMan<'a> = ChannelManager<Arc<TestChainMonitor>, Arc<TestBroadcaster>, Arc<KeyProvider>, Arc<FuzzEstimator>, &'a FuzzRouter, Arc<dyn Logger>>;

#[inline]
fn get_payment_secret_hash(dest: &ChanMan, payment_id: &mut u8) -> Option<(PaymentSecret, PaymentHash)> {
Expand Down Expand Up @@ -381,6 +399,7 @@ fn send_hop_payment(source: &ChanMan, middle: &ChanMan, middle_chan_id: u64, des
pub fn do_test<Out: Output>(data: &[u8], underlying_out: Out) {
let out = SearchingOutput::new(underlying_out);
let broadcast = Arc::new(TestBroadcaster{});
let router = FuzzRouter {};

macro_rules! make_node {
($node_id: expr, $fee_estimator: expr) => { {
Expand All @@ -399,7 +418,7 @@ pub fn do_test<Out: Output>(data: &[u8], underlying_out: Out) {
network,
best_block: BestBlock::from_genesis(network),
};
(ChannelManager::new($fee_estimator.clone(), monitor.clone(), broadcast.clone(), Arc::clone(&logger), keys_manager.clone(), config, params),
(ChannelManager::new($fee_estimator.clone(), monitor.clone(), broadcast.clone(), &router, Arc::clone(&logger), keys_manager.clone(), config, params),
monitor, keys_manager)
} }
}
Expand Down Expand Up @@ -433,6 +452,7 @@ pub fn do_test<Out: Output>(data: &[u8], underlying_out: Out) {
fee_estimator: $fee_estimator.clone(),
chain_monitor: chain_monitor.clone(),
tx_broadcaster: broadcast.clone(),
router: &router,
logger,
default_config: config,
channel_monitors: monitor_refs,
Expand Down
37 changes: 28 additions & 9 deletions fuzz/src/full_stack.rs
Original file line number Diff line number Diff line change
Expand Up @@ -35,12 +35,12 @@ use lightning::chain::chainmonitor;
use lightning::chain::transaction::OutPoint;
use lightning::chain::keysinterface::{InMemorySigner, Recipient, KeyMaterial, KeysInterface, EntropySource, NodeSigner, SignerProvider};
use lightning::ln::{PaymentHash, PaymentPreimage, PaymentSecret};
use lightning::ln::channelmanager::{ChainParameters, ChannelManager, PaymentId};
use lightning::ln::channelmanager::{ChainParameters, ChannelDetails, ChannelManager, PaymentId};
use lightning::ln::peer_handler::{MessageHandler,PeerManager,SocketDescriptor,IgnoringMessageHandler};
use lightning::ln::msgs::DecodeError;
use lightning::ln::msgs::{self, DecodeError};
use lightning::ln::script::ShutdownScript;
use lightning::routing::gossip::{P2PGossipSync, NetworkGraph};
use lightning::routing::router::{find_route, PaymentParameters, RouteParameters};
use lightning::routing::router::{find_route, InFlightHtlcs, PaymentParameters, Route, RouteHop, RouteParameters, Router};
use lightning::routing::scoring::FixedPenaltyScorer;
use lightning::util::config::UserConfig;
use lightning::util::errors::APIError;
Expand Down Expand Up @@ -127,6 +127,24 @@ impl FeeEstimator for FuzzEstimator {
}
}

struct FuzzRouter {}

impl Router for FuzzRouter {
fn find_route(
&self, _payer: &PublicKey, _params: &RouteParameters, _first_hops: Option<&[&ChannelDetails]>,
_inflight_htlcs: &InFlightHtlcs
) -> Result<Route, msgs::LightningError> {
Err(msgs::LightningError {
err: String::from("Not implemented"),
action: msgs::ErrorAction::IgnoreError
})
}
fn notify_payment_path_failed(&self, _path: &[&RouteHop], _short_channel_id: u64) {}
fn notify_payment_path_successful(&self, _path: &[&RouteHop]) {}
fn notify_payment_probe_successful(&self, _path: &[&RouteHop]) {}
fn notify_payment_probe_failed(&self, _path: &[&RouteHop], _short_channel_id: u64) {}
}

struct TestBroadcaster {
txn_broadcasted: Mutex<Vec<Transaction>>,
}
Expand Down Expand Up @@ -162,13 +180,13 @@ impl<'a> std::hash::Hash for Peer<'a> {
}
}

type ChannelMan = ChannelManager<
type ChannelMan<'a> = ChannelManager<
Arc<chainmonitor::ChainMonitor<EnforcingSigner, Arc<dyn chain::Filter>, Arc<TestBroadcaster>, Arc<FuzzEstimator>, Arc<dyn Logger>, Arc<TestPersister>>>,
Arc<TestBroadcaster>, Arc<KeyProvider>, Arc<FuzzEstimator>, Arc<dyn Logger>>;
type PeerMan<'a> = PeerManager<Peer<'a>, Arc<ChannelMan>, Arc<P2PGossipSync<Arc<NetworkGraph<Arc<dyn Logger>>>, Arc<dyn chain::Access>, Arc<dyn Logger>>>, IgnoringMessageHandler, Arc<dyn Logger>, IgnoringMessageHandler>;
Arc<TestBroadcaster>, Arc<KeyProvider>, Arc<FuzzEstimator>, &'a FuzzRouter, Arc<dyn Logger>>;
type PeerMan<'a> = PeerManager<Peer<'a>, Arc<ChannelMan<'a>>, Arc<P2PGossipSync<Arc<NetworkGraph<Arc<dyn Logger>>>, Arc<dyn chain::Access>, Arc<dyn Logger>>>, IgnoringMessageHandler, Arc<dyn Logger>, IgnoringMessageHandler>;

struct MoneyLossDetector<'a> {
manager: Arc<ChannelMan>,
manager: Arc<ChannelMan<'a>>,
monitor: Arc<chainmonitor::ChainMonitor<EnforcingSigner, Arc<dyn chain::Filter>, Arc<TestBroadcaster>, Arc<FuzzEstimator>, Arc<dyn Logger>, Arc<TestPersister>>>,
handler: PeerMan<'a>,

Expand All @@ -182,7 +200,7 @@ struct MoneyLossDetector<'a> {
}
impl<'a> MoneyLossDetector<'a> {
pub fn new(peers: &'a RefCell<[bool; 256]>,
manager: Arc<ChannelMan>,
manager: Arc<ChannelMan<'a>>,
monitor: Arc<chainmonitor::ChainMonitor<EnforcingSigner, Arc<dyn chain::Filter>, Arc<TestBroadcaster>, Arc<FuzzEstimator>, Arc<dyn Logger>, Arc<TestPersister>>>,
handler: PeerMan<'a>) -> Self {
MoneyLossDetector {
Expand Down Expand Up @@ -380,6 +398,7 @@ pub fn do_test(data: &[u8], logger: &Arc<dyn Logger>) {
let fee_est = Arc::new(FuzzEstimator {
input: input.clone(),
});
let router = FuzzRouter {};

macro_rules! get_slice {
($len: expr) => {
Expand Down Expand Up @@ -424,7 +443,7 @@ pub fn do_test(data: &[u8], logger: &Arc<dyn Logger>) {
network,
best_block: BestBlock::from_genesis(network),
};
let channelmanager = Arc::new(ChannelManager::new(fee_est.clone(), monitor.clone(), broadcast.clone(), Arc::clone(&logger), keys_manager.clone(), config, params));
let channelmanager = Arc::new(ChannelManager::new(fee_est.clone(), monitor.clone(), broadcast.clone(), &router, Arc::clone(&logger), keys_manager.clone(), config, params));
// Adding new calls to `KeysInterface::get_secure_random_bytes` during startup can change all the
// keys subsequently generated in this test. Rather than regenerating all the messages manually,
// it's easier to just increment the counter here so the keys don't change.
Expand Down
31 changes: 19 additions & 12 deletions lightning-background-processor/src/lib.rs
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,7 @@ use lightning::ln::channelmanager::ChannelManager;
use lightning::ln::msgs::{ChannelMessageHandler, OnionMessageHandler, RoutingMessageHandler};
use lightning::ln::peer_handler::{CustomMessageHandler, PeerManager, SocketDescriptor};
use lightning::routing::gossip::{NetworkGraph, P2PGossipSync};
use lightning::routing::router::Router;
use lightning::routing::scoring::WriteableScore;
use lightning::util::events::{Event, EventHandler, EventsProvider};
use lightning::util::logger::Logger;
Expand Down Expand Up @@ -342,6 +343,7 @@ pub async fn process_events_async<
T: 'static + Deref + Send + Sync,
K: 'static + Deref + Send + Sync,
F: 'static + Deref + Send + Sync,
R: 'static + Deref + Send + Sync,
G: 'static + Deref<Target = NetworkGraph<L>> + Send + Sync,
L: 'static + Deref + Send + Sync,
P: 'static + Deref + Send + Sync,
Expand All @@ -353,7 +355,7 @@ pub async fn process_events_async<
EventHandler: Fn(Event) -> EventHandlerFuture,
PS: 'static + Deref + Send,
M: 'static + Deref<Target = ChainMonitor<<K::Target as SignerProvider>::Signer, CF, T, F, L, P>> + Send + Sync,
CM: 'static + Deref<Target = ChannelManager<CW, T, K, F, L>> + Send + Sync,
CM: 'static + Deref<Target = ChannelManager<CW, T, K, F, R, L>> + Send + Sync,
PGS: 'static + Deref<Target = P2PGossipSync<G, CA, L>> + Send + Sync,
RGS: 'static + Deref<Target = RapidGossipSync<G, L>> + Send,
UMH: 'static + Deref + Send + Sync,
Expand All @@ -374,13 +376,14 @@ where
T::Target: 'static + BroadcasterInterface,
K::Target: 'static + KeysInterface,
F::Target: 'static + FeeEstimator,
R::Target: 'static + Router,
L::Target: 'static + Logger,
P::Target: 'static + Persist<<K::Target as SignerProvider>::Signer>,
CMH::Target: 'static + ChannelMessageHandler,
OMH::Target: 'static + OnionMessageHandler,
RMH::Target: 'static + RoutingMessageHandler,
UMH::Target: 'static + CustomMessageHandler,
PS::Target: 'static + Persister<'a, CW, T, K, F, L, SC>,
PS::Target: 'static + Persister<'a, CW, T, K, F, R, L, SC>,
{
let mut should_break = true;
let async_event_handler = |event| {
Expand Down Expand Up @@ -460,6 +463,7 @@ impl BackgroundProcessor {
T: 'static + Deref + Send + Sync,
K: 'static + Deref + Send + Sync,
F: 'static + Deref + Send + Sync,
R: 'static + Deref + Send + Sync,
G: 'static + Deref<Target = NetworkGraph<L>> + Send + Sync,
L: 'static + Deref + Send + Sync,
P: 'static + Deref + Send + Sync,
Expand All @@ -470,7 +474,7 @@ impl BackgroundProcessor {
EH: 'static + EventHandler + Send,
PS: 'static + Deref + Send,
M: 'static + Deref<Target = ChainMonitor<<K::Target as SignerProvider>::Signer, CF, T, F, L, P>> + Send + Sync,
CM: 'static + Deref<Target = ChannelManager<CW, T, K, F, L>> + Send + Sync,
CM: 'static + Deref<Target = ChannelManager<CW, T, K, F, R, L>> + Send + Sync,
PGS: 'static + Deref<Target = P2PGossipSync<G, CA, L>> + Send + Sync,
RGS: 'static + Deref<Target = RapidGossipSync<G, L>> + Send,
UMH: 'static + Deref + Send + Sync,
Expand All @@ -488,13 +492,14 @@ impl BackgroundProcessor {
T::Target: 'static + BroadcasterInterface,
K::Target: 'static + KeysInterface,
F::Target: 'static + FeeEstimator,
R::Target: 'static + Router,
L::Target: 'static + Logger,
P::Target: 'static + Persist<<K::Target as SignerProvider>::Signer>,
CMH::Target: 'static + ChannelMessageHandler,
OMH::Target: 'static + OnionMessageHandler,
RMH::Target: 'static + RoutingMessageHandler,
UMH::Target: 'static + CustomMessageHandler,
PS::Target: 'static + Persister<'a, CW, T, K, F, L, SC>,
PS::Target: 'static + Persister<'a, CW, T, K, F, R, L, SC>,
{
let stop_thread = Arc::new(AtomicBool::new(false));
let stop_thread_clone = stop_thread.clone();
Expand Down Expand Up @@ -579,6 +584,7 @@ mod tests {
use lightning::ln::peer_handler::{PeerManager, MessageHandler, SocketDescriptor, IgnoringMessageHandler};
use lightning::routing::gossip::{NetworkGraph, P2PGossipSync};
use lightning::routing::router::DefaultRouter;
use lightning::routing::scoring::{ProbabilisticScoringParameters, ProbabilisticScorer};
use lightning::util::config::UserConfig;
use lightning::util::events::{Event, MessageSendEventsProvider, MessageSendEvent};
use lightning::util::ser::Writeable;
Expand All @@ -593,7 +599,6 @@ mod tests {
use std::time::Duration;
use bitcoin::hashes::Hash;
use bitcoin::TxMerkleNode;
use lightning::routing::scoring::{FixedPenaltyScorer};
use lightning_rapid_gossip_sync::RapidGossipSync;
use super::{BackgroundProcessor, GossipSync, FRESHNESS_TIMER};

Expand Down Expand Up @@ -625,7 +630,7 @@ mod tests {
network_graph: Arc<NetworkGraph<Arc<test_utils::TestLogger>>>,
logger: Arc<test_utils::TestLogger>,
best_block: BestBlock,
scorer: Arc<Mutex<FixedPenaltyScorer>>,
scorer: Arc<Mutex<ProbabilisticScorer<Arc<NetworkGraph<Arc<test_utils::TestLogger>>>, Arc<test_utils::TestLogger>>>>,
}

impl Node {
Expand Down Expand Up @@ -722,24 +727,26 @@ mod tests {
for i in 0..num_nodes {
let tx_broadcaster = Arc::new(test_utils::TestBroadcaster{txn_broadcasted: Mutex::new(Vec::new()), blocks: Arc::new(Mutex::new(Vec::new()))});
let fee_estimator = Arc::new(test_utils::TestFeeEstimator { sat_per_kw: Mutex::new(253) });
let chain_source = Arc::new(test_utils::TestChainSource::new(Network::Testnet));
let logger = Arc::new(test_utils::TestLogger::with_id(format!("node {}", i)));
let persister = Arc::new(FilesystemPersister::new(format!("{}_persister_{}", persist_dir, i)));
let seed = [i as u8; 32];
let network = Network::Testnet;
let genesis_block = genesis_block(network);
let network_graph = Arc::new(NetworkGraph::new(genesis_block.header.block_hash(), logger.clone()));
let params = ProbabilisticScoringParameters::default();
let scorer = Arc::new(Mutex::new(ProbabilisticScorer::new(params, network_graph.clone(), logger.clone())));
let seed = [i as u8; 32];
let router = Arc::new(DefaultRouter::new(network_graph.clone(), logger.clone(), seed, scorer.clone()));
let chain_source = Arc::new(test_utils::TestChainSource::new(Network::Testnet));
let persister = Arc::new(FilesystemPersister::new(format!("{}_persister_{}", persist_dir, i)));
let now = Duration::from_secs(genesis_block.header.time as u64);
let keys_manager = Arc::new(KeysManager::new(&seed, now.as_secs(), now.subsec_nanos()));
let chain_monitor = Arc::new(chainmonitor::ChainMonitor::new(Some(chain_source.clone()), tx_broadcaster.clone(), logger.clone(), fee_estimator.clone(), persister.clone()));
let best_block = BestBlock::from_genesis(network);
let params = ChainParameters { network, best_block };
let manager = Arc::new(ChannelManager::new(fee_estimator.clone(), chain_monitor.clone(), tx_broadcaster.clone(), logger.clone(), keys_manager.clone(), UserConfig::default(), params));
let network_graph = Arc::new(NetworkGraph::new(genesis_block.header.block_hash(), logger.clone()));
let manager = Arc::new(ChannelManager::new(fee_estimator.clone(), chain_monitor.clone(), tx_broadcaster.clone(), router.clone(), logger.clone(), keys_manager.clone(), UserConfig::default(), params));
let p2p_gossip_sync = Arc::new(P2PGossipSync::new(network_graph.clone(), Some(chain_source.clone()), logger.clone()));
let rapid_gossip_sync = Arc::new(RapidGossipSync::new(network_graph.clone()));
let msg_handler = MessageHandler { chan_handler: Arc::new(test_utils::TestChannelMessageHandler::new()), route_handler: Arc::new(test_utils::TestRoutingMessageHandler::new()), onion_message_handler: IgnoringMessageHandler{}};
let peer_manager = Arc::new(PeerManager::new(msg_handler, keys_manager.get_node_secret(Recipient::Node).unwrap(), 0, &seed, logger.clone(), IgnoringMessageHandler{}));
let scorer = Arc::new(Mutex::new(test_utils::TestScorer::with_penalty(0)));
let node = Node { node: manager, p2p_gossip_sync, rapid_gossip_sync, peer_manager, chain_monitor, persister, tx_broadcaster, network_graph, logger, best_block, scorer };
nodes.push(node);
}
Expand Down
9 changes: 6 additions & 3 deletions lightning-block-sync/src/init.rs
Original file line number Diff line number Diff line change
Expand Up @@ -49,8 +49,8 @@ BlockSourceResult<ValidatedBlockHeader> where B::Target: BlockSource {
/// use lightning::chain::chaininterface::FeeEstimator;
/// use lightning::chain::keysinterface;
/// use lightning::chain::keysinterface::KeysInterface;
/// use lightning::ln::channelmanager::ChannelManager;
/// use lightning::ln::channelmanager::ChannelManagerReadArgs;
/// use lightning::ln::channelmanager::{ChannelManager, ChannelManagerReadArgs};
/// use lightning::routing::router::Router;
/// use lightning::util::config::UserConfig;
/// use lightning::util::logger::Logger;
/// use lightning::util::ser::ReadableArgs;
Expand All @@ -64,6 +64,7 @@ BlockSourceResult<ValidatedBlockHeader> where B::Target: BlockSource {
/// K: KeysInterface,
/// T: BroadcasterInterface,
/// F: FeeEstimator,
/// R: Router,
/// L: Logger,
/// C: chain::Filter,
/// P: chainmonitor::Persist<K::Signer>,
Expand All @@ -74,6 +75,7 @@ BlockSourceResult<ValidatedBlockHeader> where B::Target: BlockSource {
/// keys_manager: &K,
/// tx_broadcaster: &T,
/// fee_estimator: &F,
/// router: &R,
/// logger: &L,
/// persister: &P,
/// ) {
Expand All @@ -90,11 +92,12 @@ BlockSourceResult<ValidatedBlockHeader> where B::Target: BlockSource {
/// fee_estimator,
/// chain_monitor,
/// tx_broadcaster,
/// router,
/// logger,
/// config,
/// vec![&mut monitor],
/// );
/// <(BlockHash, ChannelManager<&ChainMonitor<K::Signer, &C, &T, &F, &L, &P>, &T, &K, &F, &L>)>::read(
/// <(BlockHash, ChannelManager<&ChainMonitor<K::Signer, &C, &T, &F, &L, &P>, &T, &K, &F, &R, &L>)>::read(
/// &mut Cursor::new(&serialized_manager), read_args).unwrap()
/// };
///
Expand Down
Loading

0 comments on commit b79ff71

Please sign in to comment.