refactor(swarm)!: remove handler from NetworkBehaviourAction::Dial (#3328)

We create the `ConnectionId` for the new connection as part of `DialOpts`. This allows `NetworkBehaviour`s to accurately track state regarding their own dial attempts.

This patch is the main enabler of https://github.com/libp2p/rust-libp2p/pull/3254. Removing the `handler` field will allow us to deprecate the `NetworkBehaviour::new_handler` function in favor of four new ones that give more control over the connection lifecycle.
This commit is contained in:
Thomas Eizinger
2023-02-14 14:09:29 +13:00
committed by GitHub
parent 9247cfa878
commit caed1fe2c7
31 changed files with 584 additions and 967 deletions

View File

@ -28,15 +28,13 @@ pub use listen_addresses::ListenAddresses;
use crate::connection::ConnectionId;
use crate::dial_opts::DialOpts;
use crate::handler::{ConnectionHandler, IntoConnectionHandler};
use crate::{AddressRecord, AddressScore, DialError, ListenError, THandlerOutEvent};
use crate::handler::IntoConnectionHandler;
use crate::{
AddressRecord, AddressScore, DialError, ListenError, THandlerInEvent, THandlerOutEvent,
};
use libp2p_core::{transport::ListenerId, ConnectedPoint, Multiaddr, PeerId};
use std::{task::Context, task::Poll};
/// Custom event that can be received by the [`ConnectionHandler`].
pub(crate) type THandlerInEvent<THandler> =
<<THandler as IntoConnectionHandler>::Handler as ConnectionHandler>::InEvent;
/// A [`NetworkBehaviour`] defines the behaviour of the local node on the network.
///
/// In contrast to [`Transport`](libp2p_core::Transport) which defines **how** to send bytes on the
@ -126,7 +124,7 @@ pub trait NetworkBehaviour: 'static {
/// Event generated by the `NetworkBehaviour` and that the swarm will report back.
type OutEvent: Send + 'static;
/// Creates a new [`ConnectionHandler`] for a connection with a peer.
/// Creates a new [`ConnectionHandler`](crate::ConnectionHandler) for a connection with a peer.
///
/// Every time an incoming connection is opened, and every time another [`NetworkBehaviour`]
/// emitted a dial request, this method is called.
@ -158,8 +156,8 @@ pub trait NetworkBehaviour: 'static {
/// Informs the behaviour about an event from the [`Swarm`](crate::Swarm).
fn on_swarm_event(&mut self, event: FromSwarm<Self::ConnectionHandler>);
/// Informs the behaviour about an event generated by the [`ConnectionHandler`] dedicated to the
/// peer identified by `peer_id`. for the behaviour.
/// Informs the behaviour about an event generated by the [`ConnectionHandler`](crate::ConnectionHandler)
/// dedicated to the peer identified by `peer_id`. for the behaviour.
///
/// The [`PeerId`] is guaranteed to be in a connected state. In other words,
/// [`FromSwarm::ConnectionEstablished`] has previously been received with this [`PeerId`].
@ -178,7 +176,7 @@ pub trait NetworkBehaviour: 'static {
&mut self,
cx: &mut Context<'_>,
params: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>>;
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>>;
}
/// Parameters passed to `poll()`, that the `NetworkBehaviour` has access to.
@ -224,16 +222,8 @@ pub trait PollParameters {
/// in whose context it is executing.
///
/// [`Swarm`]: super::Swarm
//
// Note: `TInEvent` is needed to be able to implement
// [`NetworkBehaviourAction::map_in`], mapping the handler `InEvent` leaving the
// handler itself untouched.
#[derive(Debug)]
pub enum NetworkBehaviourAction<
TOutEvent,
THandler: IntoConnectionHandler,
TInEvent = THandlerInEvent<THandler>,
> {
pub enum NetworkBehaviourAction<TOutEvent, TInEvent> {
/// Instructs the `Swarm` to return an event when it is being polled.
GenerateEvent(TOutEvent),
@ -242,195 +232,28 @@ pub enum NetworkBehaviourAction<
/// On success, [`NetworkBehaviour::on_swarm_event`] with `ConnectionEstablished` is invoked.
/// On failure, [`NetworkBehaviour::on_swarm_event`] with `DialFailure` is invoked.
///
/// Note that the provided handler is returned to the [`NetworkBehaviour`] on connection failure
/// and connection closing. Thus it can be used to carry state, which otherwise would have to be
/// tracked in the [`NetworkBehaviour`] itself. E.g. a message destined to an unconnected peer
/// can be included in the handler, and thus directly send on connection success or extracted by
/// the [`NetworkBehaviour`] on connection failure.
///
/// # Example carrying state in the handler
///
/// ```rust
/// # use futures::executor::block_on;
/// # use futures::stream::StreamExt;
/// # use libp2p_core::identity;
/// # use libp2p_core::transport::{MemoryTransport, Transport};
/// # use libp2p_core::upgrade::{self, DeniedUpgrade, InboundUpgrade, OutboundUpgrade};
/// # use libp2p_core::PeerId;
/// # use libp2p_plaintext::PlainText2Config;
/// # use libp2p_swarm::{
/// # ConnectionId, DialError, IntoConnectionHandler, KeepAlive, NegotiatedSubstream,
/// # FromSwarm, DialFailure, THandlerOutEvent,
/// # NetworkBehaviour, NetworkBehaviourAction, PollParameters, ConnectionHandler,
/// # ConnectionHandlerEvent, ConnectionHandlerUpgrErr, SubstreamProtocol, Swarm, SwarmEvent,
/// # };
/// # use libp2p_swarm::handler::ConnectionEvent;
/// # use libp2p_swarm::dial_opts::{DialOpts, PeerCondition};
/// # use libp2p_yamux as yamux;
/// # use std::collections::VecDeque;
/// # use std::task::{Context, Poll};
/// # use void::Void;
/// #
/// # let local_key = identity::Keypair::generate_ed25519();
/// # let local_public_key = local_key.public();
/// # let local_peer_id = PeerId::from(local_public_key.clone());
/// #
/// # let transport = MemoryTransport::default()
/// # .upgrade(upgrade::Version::V1)
/// # .authenticate(PlainText2Config { local_public_key })
/// # .multiplex(yamux::YamuxConfig::default())
/// # .boxed();
/// #
/// # let mut swarm = Swarm::with_threadpool_executor(transport, MyBehaviour::default(), local_peer_id);
/// #
/// // Super precious message that we should better not lose.
/// let message = PreciousMessage("My precious message".to_string());
///
/// // Unfortunately this peer is offline, thus sending our message to it will fail.
/// let offline_peer = PeerId::random();
///
/// // Let's send it anyways. We should get it back in case connecting to the peer fails.
/// swarm.behaviour_mut().send(offline_peer, message);
///
/// block_on(async {
/// // As expected, sending failed. But great news, we got our message back.
/// matches!(
/// swarm.next().await.expect("Infinite stream"),
/// SwarmEvent::Behaviour(PreciousMessage(_))
/// );
/// });
///
/// #[derive(Default)]
/// struct MyBehaviour {
/// outbox_to_swarm: VecDeque<NetworkBehaviourAction<PreciousMessage, MyHandler>>,
/// }
///
/// impl MyBehaviour {
/// fn send(&mut self, peer_id: PeerId, msg: PreciousMessage) {
/// self.outbox_to_swarm
/// .push_back(NetworkBehaviourAction::Dial {
/// opts: DialOpts::peer_id(peer_id)
/// .condition(PeerCondition::Always)
/// .build(),
/// handler: MyHandler { message: Some(msg) },
/// });
/// }
/// }
/// #
/// impl NetworkBehaviour for MyBehaviour {
/// # type ConnectionHandler = MyHandler;
/// # type OutEvent = PreciousMessage;
/// #
/// # fn new_handler(&mut self) -> Self::ConnectionHandler {
/// # MyHandler { message: None }
/// # }
/// #
/// #
/// # fn on_connection_handler_event(
/// # &mut self,
/// # _: PeerId,
/// # _: ConnectionId,
/// # _: THandlerOutEvent<Self>,
/// # ) {
/// # unreachable!();
/// # }
/// #
/// fn on_swarm_event(
/// &mut self,
/// event: FromSwarm<Self::ConnectionHandler>,
/// ) {
/// // As expected, sending the message failed. But lucky us, we got the handler back, thus
/// // the precious message is not lost and we can return it back to the user.
/// if let FromSwarm::DialFailure(DialFailure { handler, .. }) = event {
/// let msg = handler.message.unwrap();
/// self.outbox_to_swarm
/// .push_back(NetworkBehaviourAction::GenerateEvent(msg))
/// }
/// }
/// #
/// # fn poll(
/// # &mut self,
/// # _: &mut Context<'_>,
/// # _: &mut impl PollParameters,
/// # ) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
/// # if let Some(action) = self.outbox_to_swarm.pop_front() {
/// # return Poll::Ready(action);
/// # }
/// # Poll::Pending
/// # }
/// }
///
/// # struct MyHandler {
/// # message: Option<PreciousMessage>,
/// # }
/// #
/// # impl ConnectionHandler for MyHandler {
/// # type InEvent = Void;
/// # type OutEvent = Void;
/// # type Error = Void;
/// # type InboundProtocol = DeniedUpgrade;
/// # type OutboundProtocol = DeniedUpgrade;
/// # type InboundOpenInfo = ();
/// # type OutboundOpenInfo = Void;
/// #
/// # fn listen_protocol(
/// # &self,
/// # ) -> SubstreamProtocol<Self::InboundProtocol, Self::InboundOpenInfo> {
/// # SubstreamProtocol::new(DeniedUpgrade, ())
/// # }
/// #
/// # fn on_behaviour_event(&mut self, _event: Self::InEvent) {}
/// #
/// # fn on_connection_event(
/// # &mut self,
/// # event: ConnectionEvent<
/// # Self::InboundProtocol,
/// # Self::OutboundProtocol,
/// # Self::InboundOpenInfo,
/// # Self::OutboundOpenInfo,
/// # >,
/// # ) {}
/// #
/// # fn connection_keep_alive(&self) -> KeepAlive {
/// # KeepAlive::Yes
/// # }
/// #
/// # fn poll(
/// # &mut self,
/// # _: &mut Context<'_>,
/// # ) -> Poll<
/// # ConnectionHandlerEvent<
/// # Self::OutboundProtocol,
/// # Self::OutboundOpenInfo,
/// # Self::OutEvent,
/// # Self::Error,
/// # >,
/// # > {
/// # todo!("If `Self::message.is_some()` send the message to the remote.")
/// # }
/// # }
/// # #[derive(Debug, PartialEq, Eq)]
/// # struct PreciousMessage(String);
/// ```
Dial { opts: DialOpts, handler: THandler },
/// [`DialOpts`] provides access to the [`ConnectionId`] via [`DialOpts::connection_id`].
/// This [`ConnectionId`] will be used throughout the connection's lifecycle to associate events with it.
/// This allows a [`NetworkBehaviour`] to identify a connection that resulted out of its own dial request.
Dial { opts: DialOpts },
/// Instructs the `Swarm` to send an event to the handler dedicated to a
/// connection with a peer.
///
/// If the `Swarm` is connected to the peer, the message is delivered to the
/// [`ConnectionHandler`] instance identified by the peer ID and connection ID.
/// [`ConnectionHandler`](crate::ConnectionHandler) instance identified by the peer ID and connection ID.
///
/// If the specified connection no longer exists, the event is silently dropped.
///
/// Typically the connection ID given is the same as the one passed to
/// [`NetworkBehaviour::on_connection_handler_event`], i.e. whenever the behaviour wishes to
/// respond to a request on the same connection (and possibly the same
/// substream, as per the implementation of [`ConnectionHandler`]).
/// substream, as per the implementation of [`ConnectionHandler`](crate::ConnectionHandler)).
///
/// Note that even if the peer is currently connected, connections can get closed
/// at any time and thus the event may not reach a handler.
NotifyHandler {
/// The peer for whom a [`ConnectionHandler`] should be notified.
/// The peer for whom a [`ConnectionHandler`](crate::ConnectionHandler) should be notified.
peer_id: PeerId,
/// The options w.r.t. which connection handler to notify of the event.
handler: NotifyHandler,
@ -459,10 +282,10 @@ pub enum NetworkBehaviourAction<
///
/// Note: Closing a connection via
/// [`NetworkBehaviourAction::CloseConnection`] does not inform the
/// corresponding [`ConnectionHandler`].
/// Closing a connection via a [`ConnectionHandler`] can be done
/// either in a collaborative manner across [`ConnectionHandler`]s
/// with [`ConnectionHandler::connection_keep_alive`] or directly with
/// corresponding [`ConnectionHandler`](crate::ConnectionHandler).
/// Closing a connection via a [`ConnectionHandler`](crate::ConnectionHandler) can be done
/// either in a collaborative manner across [`ConnectionHandler`](crate::ConnectionHandler)s
/// with [`ConnectionHandler::connection_keep_alive`](crate::ConnectionHandler::connection_keep_alive) or directly with
/// [`ConnectionHandlerEvent::Close`](crate::ConnectionHandlerEvent::Close).
CloseConnection {
/// The peer to disconnect.
@ -472,19 +295,15 @@ pub enum NetworkBehaviourAction<
},
}
impl<TOutEvent, THandler: IntoConnectionHandler, TInEventOld>
NetworkBehaviourAction<TOutEvent, THandler, TInEventOld>
{
impl<TOutEvent, TInEventOld> NetworkBehaviourAction<TOutEvent, TInEventOld> {
/// Map the handler event.
pub fn map_in<TInEventNew>(
self,
f: impl FnOnce(TInEventOld) -> TInEventNew,
) -> NetworkBehaviourAction<TOutEvent, THandler, TInEventNew> {
) -> NetworkBehaviourAction<TOutEvent, TInEventNew> {
match self {
NetworkBehaviourAction::GenerateEvent(e) => NetworkBehaviourAction::GenerateEvent(e),
NetworkBehaviourAction::Dial { opts, handler } => {
NetworkBehaviourAction::Dial { opts, handler }
}
NetworkBehaviourAction::Dial { opts } => NetworkBehaviourAction::Dial { opts },
NetworkBehaviourAction::NotifyHandler {
peer_id,
handler,
@ -508,14 +327,15 @@ impl<TOutEvent, THandler: IntoConnectionHandler, TInEventOld>
}
}
impl<TOutEvent, THandler: IntoConnectionHandler> NetworkBehaviourAction<TOutEvent, THandler> {
impl<TOutEvent, THandlerIn> NetworkBehaviourAction<TOutEvent, THandlerIn> {
/// Map the event the swarm will return.
pub fn map_out<E>(self, f: impl FnOnce(TOutEvent) -> E) -> NetworkBehaviourAction<E, THandler> {
pub fn map_out<E>(
self,
f: impl FnOnce(TOutEvent) -> E,
) -> NetworkBehaviourAction<E, THandlerIn> {
match self {
NetworkBehaviourAction::GenerateEvent(e) => NetworkBehaviourAction::GenerateEvent(f(e)),
NetworkBehaviourAction::Dial { opts, handler } => {
NetworkBehaviourAction::Dial { opts, handler }
}
NetworkBehaviourAction::Dial { opts } => NetworkBehaviourAction::Dial { opts },
NetworkBehaviourAction::NotifyHandler {
peer_id,
handler,
@ -539,93 +359,6 @@ impl<TOutEvent, THandler: IntoConnectionHandler> NetworkBehaviourAction<TOutEven
}
}
impl<TInEvent, TOutEvent, THandlerOld> NetworkBehaviourAction<TOutEvent, THandlerOld>
where
THandlerOld: IntoConnectionHandler,
<THandlerOld as IntoConnectionHandler>::Handler: ConnectionHandler<InEvent = TInEvent>,
{
/// Map the handler.
pub fn map_handler<THandlerNew>(
self,
f: impl FnOnce(THandlerOld) -> THandlerNew,
) -> NetworkBehaviourAction<TOutEvent, THandlerNew>
where
THandlerNew: IntoConnectionHandler,
<THandlerNew as IntoConnectionHandler>::Handler: ConnectionHandler<InEvent = TInEvent>,
{
match self {
NetworkBehaviourAction::GenerateEvent(e) => NetworkBehaviourAction::GenerateEvent(e),
NetworkBehaviourAction::Dial { opts, handler } => NetworkBehaviourAction::Dial {
opts,
handler: f(handler),
},
NetworkBehaviourAction::NotifyHandler {
peer_id,
handler,
event,
} => NetworkBehaviourAction::NotifyHandler {
peer_id,
handler,
event,
},
NetworkBehaviourAction::ReportObservedAddr { address, score } => {
NetworkBehaviourAction::ReportObservedAddr { address, score }
}
NetworkBehaviourAction::CloseConnection {
peer_id,
connection,
} => NetworkBehaviourAction::CloseConnection {
peer_id,
connection,
},
}
}
}
impl<TInEventOld, TOutEvent, THandlerOld> NetworkBehaviourAction<TOutEvent, THandlerOld>
where
THandlerOld: IntoConnectionHandler,
<THandlerOld as IntoConnectionHandler>::Handler: ConnectionHandler<InEvent = TInEventOld>,
{
/// Map the handler and handler event.
pub fn map_handler_and_in<THandlerNew, TInEventNew>(
self,
f_handler: impl FnOnce(THandlerOld) -> THandlerNew,
f_in_event: impl FnOnce(TInEventOld) -> TInEventNew,
) -> NetworkBehaviourAction<TOutEvent, THandlerNew>
where
THandlerNew: IntoConnectionHandler,
<THandlerNew as IntoConnectionHandler>::Handler: ConnectionHandler<InEvent = TInEventNew>,
{
match self {
NetworkBehaviourAction::GenerateEvent(e) => NetworkBehaviourAction::GenerateEvent(e),
NetworkBehaviourAction::Dial { opts, handler } => NetworkBehaviourAction::Dial {
opts,
handler: f_handler(handler),
},
NetworkBehaviourAction::NotifyHandler {
peer_id,
handler,
event,
} => NetworkBehaviourAction::NotifyHandler {
peer_id,
handler,
event: f_in_event(event),
},
NetworkBehaviourAction::ReportObservedAddr { address, score } => {
NetworkBehaviourAction::ReportObservedAddr { address, score }
}
NetworkBehaviourAction::CloseConnection {
peer_id,
connection,
} => NetworkBehaviourAction::CloseConnection {
peer_id,
connection,
},
}
}
}
/// The options w.r.t. which connection handler to notify of an event.
#[derive(Debug, Clone)]
pub enum NotifyHandler {
@ -661,13 +394,13 @@ pub enum FromSwarm<'a, Handler: IntoConnectionHandler> {
AddressChange(AddressChange<'a>),
/// Informs the behaviour that the dial to a known
/// or unknown node failed.
DialFailure(DialFailure<'a, Handler>),
DialFailure(DialFailure<'a>),
/// Informs the behaviour that an error
/// happened on an incoming connection during its initial handshake.
///
/// This can include, for example, an error during the handshake of the encryption layer, or the
/// connection unexpectedly closed.
ListenFailure(ListenFailure<'a, Handler>),
ListenFailure(ListenFailure<'a>),
/// Informs the behaviour that a new listener was created.
NewListener(NewListener),
/// Informs the behaviour that we have started listening on a new multiaddr.
@ -722,10 +455,10 @@ pub struct AddressChange<'a> {
/// [`FromSwarm`] variant that informs the behaviour that the dial to a known
/// or unknown node failed.
#[derive(Clone, Copy)]
pub struct DialFailure<'a, Handler> {
pub struct DialFailure<'a> {
pub peer_id: Option<PeerId>,
pub handler: Handler,
pub error: &'a DialError,
pub connection_id: ConnectionId,
}
/// [`FromSwarm`] variant that informs the behaviour that an error
@ -734,11 +467,11 @@ pub struct DialFailure<'a, Handler> {
/// This can include, for example, an error during the handshake of the encryption layer, or the
/// connection unexpectedly closed.
#[derive(Clone, Copy)]
pub struct ListenFailure<'a, Handler> {
pub struct ListenFailure<'a> {
pub local_addr: &'a Multiaddr,
pub send_back_addr: &'a Multiaddr,
pub error: &'a ListenError,
pub handler: Handler,
pub connection_id: ConnectionId,
}
/// [`FromSwarm`] variant that informs the behaviour that a new listener was created.
@ -794,7 +527,6 @@ pub struct ExpiredExternalAddr<'a> {
impl<'a, Handler: IntoConnectionHandler> FromSwarm<'a, Handler> {
fn map_handler<NewHandler>(
self,
map_into_handler: impl FnOnce(Handler) -> NewHandler,
map_handler: impl FnOnce(
<Handler as IntoConnectionHandler>::Handler,
) -> <NewHandler as IntoConnectionHandler>::Handler,
@ -802,13 +534,12 @@ impl<'a, Handler: IntoConnectionHandler> FromSwarm<'a, Handler> {
where
NewHandler: IntoConnectionHandler,
{
self.maybe_map_handler(|h| Some(map_into_handler(h)), |h| Some(map_handler(h)))
self.maybe_map_handler(|h| Some(map_handler(h)))
.expect("To return Some as all closures return Some.")
}
fn maybe_map_handler<NewHandler>(
self,
map_into_handler: impl FnOnce(Handler) -> Option<NewHandler>,
map_handler: impl FnOnce(
<Handler as IntoConnectionHandler>::Handler,
) -> Option<<NewHandler as IntoConnectionHandler>::Handler>,
@ -856,23 +587,23 @@ impl<'a, Handler: IntoConnectionHandler> FromSwarm<'a, Handler> {
})),
FromSwarm::DialFailure(DialFailure {
peer_id,
handler,
error,
connection_id,
}) => Some(FromSwarm::DialFailure(DialFailure {
peer_id,
handler: map_into_handler(handler)?,
error,
connection_id,
})),
FromSwarm::ListenFailure(ListenFailure {
local_addr,
send_back_addr,
connection_id,
error,
handler,
}) => Some(FromSwarm::ListenFailure(ListenFailure {
local_addr,
send_back_addr,
connection_id,
error,
handler: map_into_handler(handler)?,
})),
FromSwarm::NewListener(NewListener { listener_id }) => {
Some(FromSwarm::NewListener(NewListener { listener_id }))

View File

@ -21,6 +21,7 @@
use crate::behaviour::{self, NetworkBehaviour, NetworkBehaviourAction, PollParameters};
use crate::connection::ConnectionId;
use crate::handler::either::IntoEitherHandler;
use crate::THandlerInEvent;
use crate::THandlerOutEvent;
use either::Either;
use libp2p_core::{Multiaddr, PeerId};
@ -51,20 +52,14 @@ where
fn on_swarm_event(&mut self, event: behaviour::FromSwarm<Self::ConnectionHandler>) {
match self {
Either::Left(b) => b.on_swarm_event(event.map_handler(
|h| h.unwrap_left(),
|h| match h {
Either::Left(h) => h,
Either::Right(_) => unreachable!(),
},
)),
Either::Right(b) => b.on_swarm_event(event.map_handler(
|h| h.unwrap_right(),
|h| match h {
Either::Right(h) => h,
Either::Left(_) => unreachable!(),
},
)),
Either::Left(b) => b.on_swarm_event(event.map_handler(|h| match h {
Either::Left(h) => h,
Either::Right(_) => unreachable!(),
})),
Either::Right(b) => b.on_swarm_event(event.map_handler(|h| match h {
Either::Right(h) => h,
Either::Left(_) => unreachable!(),
})),
}
}
@ -89,14 +84,14 @@ where
&mut self,
cx: &mut Context<'_>,
params: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>> {
let event = match self {
Either::Left(behaviour) => futures::ready!(behaviour.poll(cx, params))
.map_out(Either::Left)
.map_handler_and_in(IntoEitherHandler::Left, Either::Left),
.map_in(Either::Left),
Either::Right(behaviour) => futures::ready!(behaviour.poll(cx, params))
.map_out(Either::Right)
.map_handler_and_in(IntoEitherHandler::Right, Either::Right),
.map_in(Either::Right),
};
Poll::Ready(event)

View File

@ -26,7 +26,9 @@ use crate::handler::{
IntoConnectionHandler, KeepAlive, ListenUpgradeError, SubstreamProtocol,
};
use crate::upgrade::SendWrapper;
use crate::{NetworkBehaviour, NetworkBehaviourAction, PollParameters, THandlerOutEvent};
use crate::{
NetworkBehaviour, NetworkBehaviourAction, PollParameters, THandlerInEvent, THandlerOutEvent,
};
use either::Either;
use futures::future;
use libp2p_core::{upgrade::DeniedUpgrade, ConnectedPoint, Multiaddr, PeerId};
@ -84,7 +86,7 @@ where
fn on_swarm_event(&mut self, event: FromSwarm<Self::ConnectionHandler>) {
if let Some(behaviour) = &mut self.inner {
if let Some(event) = event.maybe_map_handler(|h| h.inner, |h| h.inner) {
if let Some(event) = event.maybe_map_handler(|h| h.inner) {
behaviour.on_swarm_event(event);
}
}
@ -105,11 +107,9 @@ where
&mut self,
cx: &mut Context<'_>,
params: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>> {
if let Some(inner) = self.inner.as_mut() {
inner.poll(cx, params).map(|action| {
action.map_handler(|h| ToggleIntoConnectionHandler { inner: Some(h) })
})
inner.poll(cx, params)
} else {
Poll::Pending
}

View File

@ -21,7 +21,6 @@
use crate::connection::{Connection, ConnectionId, PendingPoint};
use crate::{
behaviour::THandlerInEvent,
connection::{
Connected, ConnectionError, ConnectionLimit, IncomingInfo, PendingConnectionError,
PendingInboundConnectionError, PendingOutboundConnectionError,
@ -72,7 +71,9 @@ impl ExecSwitch {
}
}
fn spawn(&mut self, task: BoxFuture<'static, ()>) {
fn spawn(&mut self, task: impl Future<Output = ()> + Send + 'static) {
let task = task.boxed();
match self {
Self::Executor(executor) => executor.exec(task),
Self::LocalSpawn(local) => local.push(task),
@ -100,7 +101,7 @@ where
>,
/// The pending connections that are currently being negotiated.
pending: HashMap<ConnectionId, PendingConnection<THandler>>,
pending: HashMap<ConnectionId, PendingConnection>,
/// Size of the task command buffer (per task).
task_command_buffer_size: usize,
@ -187,11 +188,9 @@ impl<TInEvent> EstablishedConnection<TInEvent> {
}
}
struct PendingConnection<THandler> {
struct PendingConnection {
/// [`PeerId`] of the remote peer.
peer_id: Option<PeerId>,
/// Handler to handle connection once no longer pending but established.
handler: THandler,
endpoint: PendingPoint,
/// When dropped, notifies the task which then knows to terminate.
abort_notifier: Option<oneshot::Sender<Void>>,
@ -199,7 +198,7 @@ struct PendingConnection<THandler> {
accepted_at: Instant,
}
impl<THandler> PendingConnection<THandler> {
impl PendingConnection {
fn is_for_same_remote_as(&self, other: PeerId) -> bool {
self.peer_id.map_or(false, |peer| peer == other)
}
@ -228,10 +227,7 @@ pub enum PoolEvent<THandler: IntoConnectionHandler> {
id: ConnectionId,
peer_id: PeerId,
endpoint: ConnectedPoint,
/// List of other connections to the same peer.
///
/// Note: Does not include the connection reported through this event.
other_established_connection_ids: Vec<ConnectionId>,
connection: StreamMuxerBox,
/// [`Some`] when the new connection is an outgoing connection.
/// Addresses are dialed in parallel. Contains the addresses and errors
/// of dial attempts that failed before the one successful dial.
@ -269,8 +265,6 @@ pub enum PoolEvent<THandler: IntoConnectionHandler> {
id: ConnectionId,
/// The error that occurred.
error: PendingOutboundConnectionError,
/// The handler that was supposed to handle the connection.
handler: THandler,
/// The (expected) peer of the failed connection.
peer: Option<PeerId>,
},
@ -285,8 +279,6 @@ pub enum PoolEvent<THandler: IntoConnectionHandler> {
local_addr: Multiaddr,
/// The error that occurred.
error: PendingInboundConnectionError,
/// The handler that was supposed to handle the connection.
handler: THandler,
},
/// A node has produced an event.
@ -346,7 +338,11 @@ where
pub fn get_established(
&mut self,
id: ConnectionId,
) -> Option<&mut EstablishedConnection<THandlerInEvent<THandler>>> {
) -> Option<
&mut EstablishedConnection<
<<THandler as IntoConnectionHandler>::Handler as ConnectionHandler>::InEvent,
>,
> {
self.established
.values_mut()
.find_map(|connections| connections.get_mut(&id))
@ -410,15 +406,6 @@ where
self.established.keys()
}
fn spawn(&mut self, task: BoxFuture<'static, ()>) {
self.executor.spawn(task)
}
}
impl<THandler> Pool<THandler>
where
THandler: IntoConnectionHandler,
{
/// Adds a pending outgoing connection to the pool in the form of a `Future`
/// that establishes and negotiates the connection.
///
@ -436,32 +423,26 @@ where
>,
>,
peer: Option<PeerId>,
handler: THandler,
role_override: Endpoint,
dial_concurrency_factor_override: Option<NonZeroU8>,
) -> Result<ConnectionId, (ConnectionLimit, THandler)> {
if let Err(limit) = self.counters.check_max_pending_outgoing() {
return Err((limit, handler));
};
connection_id: ConnectionId,
) -> Result<(), ConnectionLimit> {
self.counters.check_max_pending_outgoing()?;
let dial = ConcurrentDial::new(
dials,
dial_concurrency_factor_override.unwrap_or(self.dial_concurrency_factor),
);
let connection_id = ConnectionId::next();
let (abort_notifier, abort_receiver) = oneshot::channel();
self.spawn(
task::new_for_pending_outgoing_connection(
self.executor
.spawn(task::new_for_pending_outgoing_connection(
connection_id,
dial,
abort_receiver,
self.pending_connection_events_tx.clone(),
)
.boxed(),
);
));
let endpoint = PendingPoint::Dialer { role_override };
@ -470,13 +451,13 @@ where
connection_id,
PendingConnection {
peer_id: peer,
handler,
endpoint,
abort_notifier: Some(abort_notifier),
accepted_at: Instant::now(),
},
);
Ok(connection_id)
Ok(())
}
/// Adds a pending incoming connection to the pool in the form of a
@ -487,44 +468,85 @@ where
pub fn add_incoming<TFut>(
&mut self,
future: TFut,
handler: THandler,
info: IncomingInfo<'_>,
) -> Result<ConnectionId, (ConnectionLimit, THandler)>
connection_id: ConnectionId,
) -> Result<(), ConnectionLimit>
where
TFut: Future<Output = Result<(PeerId, StreamMuxerBox), std::io::Error>> + Send + 'static,
{
let endpoint = info.create_connected_point();
if let Err(limit) = self.counters.check_max_pending_incoming() {
return Err((limit, handler));
}
let connection_id = ConnectionId::next();
self.counters.check_max_pending_incoming()?;
let (abort_notifier, abort_receiver) = oneshot::channel();
self.spawn(
task::new_for_pending_incoming_connection(
self.executor
.spawn(task::new_for_pending_incoming_connection(
connection_id,
future,
abort_receiver,
self.pending_connection_events_tx.clone(),
)
.boxed(),
);
));
self.counters.inc_pending_incoming();
self.pending.insert(
connection_id,
PendingConnection {
peer_id: None,
handler,
endpoint: endpoint.into(),
abort_notifier: Some(abort_notifier),
accepted_at: Instant::now(),
},
);
Ok(connection_id)
Ok(())
}
pub fn spawn_connection(
&mut self,
id: ConnectionId,
obtained_peer_id: PeerId,
endpoint: &ConnectedPoint,
muxer: StreamMuxerBox,
handler: <THandler as IntoConnectionHandler>::Handler,
) {
let conns = self.established.entry(obtained_peer_id).or_default();
self.counters.inc_established(endpoint);
let (command_sender, command_receiver) = mpsc::channel(self.task_command_buffer_size);
let (event_sender, event_receiver) = mpsc::channel(self.per_connection_event_buffer_size);
conns.insert(
id,
EstablishedConnection {
endpoint: endpoint.clone(),
sender: command_sender,
},
);
self.established_connection_events.push(event_receiver);
if let Some(waker) = self.no_established_connections_waker.take() {
waker.wake();
}
let connection = Connection::new(
muxer,
handler,
self.substream_upgrade_protocol_override,
self.max_negotiating_inbound_streams,
);
self.executor.spawn(task::new_for_established_connection(
id,
obtained_peer_id,
connection,
command_receiver,
event_sender,
))
}
pub fn close_connection(&mut self, muxer: StreamMuxerBox) {
self.executor.spawn(async move {
let _ = muxer.close().await;
});
}
/// Polls the connection pool for events.
@ -614,7 +636,6 @@ where
} => {
let PendingConnection {
peer_id: expected_peer_id,
handler,
endpoint,
abort_notifier: _,
accepted_at,
@ -695,20 +716,17 @@ where
});
if let Err(error) = error {
self.spawn(
poll_fn(move |cx| {
if let Err(e) = ready!(muxer.poll_close_unpin(cx)) {
log::debug!(
"Failed to close connection {:?} to peer {}: {:?}",
id,
obtained_peer_id,
e
);
}
Poll::Ready(())
})
.boxed(),
);
self.executor.spawn(poll_fn(move |cx| {
if let Err(e) = ready!(muxer.poll_close_unpin(cx)) {
log::debug!(
"Failed to close connection {:?} to peer {}: {:?}",
id,
obtained_peer_id,
e
);
}
Poll::Ready(())
}));
match endpoint {
ConnectedPoint::Dialer { .. } => {
@ -716,7 +734,6 @@ where
id,
error: error
.map(|t| vec![(endpoint.get_remote_address().clone(), t)]),
handler,
peer: expected_peer_id.or(Some(obtained_peer_id)),
})
}
@ -727,7 +744,6 @@ where
return Poll::Ready(PoolEvent::PendingInboundConnectionError {
id,
error,
handler,
send_back_addr,
local_addr,
})
@ -735,51 +751,13 @@ where
};
}
// Add the connection to the pool.
let conns = self.established.entry(obtained_peer_id).or_default();
let other_established_connection_ids = conns.keys().cloned().collect();
self.counters.inc_established(&endpoint);
let (command_sender, command_receiver) =
mpsc::channel(self.task_command_buffer_size);
let (event_sender, event_receiver) =
mpsc::channel(self.per_connection_event_buffer_size);
conns.insert(
id,
EstablishedConnection {
endpoint: endpoint.clone(),
sender: command_sender,
},
);
self.established_connection_events.push(event_receiver);
if let Some(waker) = self.no_established_connections_waker.take() {
waker.wake();
}
let connection = Connection::new(
muxer,
handler.into_handler(&obtained_peer_id, &endpoint),
self.substream_upgrade_protocol_override,
self.max_negotiating_inbound_streams,
);
self.spawn(
task::new_for_established_connection(
id,
obtained_peer_id,
connection,
command_receiver,
event_sender,
)
.boxed(),
);
let established_in = accepted_at.elapsed();
return Poll::Ready(PoolEvent::ConnectionEstablished {
peer_id: obtained_peer_id,
endpoint,
id,
other_established_connection_ids,
connection: muxer,
concurrent_dial_errors,
established_in,
});
@ -787,7 +765,6 @@ where
task::PendingConnectionEvent::PendingFailed { id, error } => {
if let Some(PendingConnection {
peer_id,
handler,
endpoint,
abort_notifier: _,
accepted_at: _, // Ignoring the time it took for the connection to fail.
@ -800,7 +777,6 @@ where
return Poll::Ready(PoolEvent::PendingOutboundConnectionError {
id,
error,
handler,
peer: peer_id,
});
}
@ -814,7 +790,6 @@ where
return Poll::Ready(PoolEvent::PendingInboundConnectionError {
id,
error,
handler,
send_back_addr,
local_addr,
});

View File

@ -19,6 +19,7 @@
// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER
// DEALINGS IN THE SOFTWARE.
use crate::ConnectionId;
use libp2p_core::connection::Endpoint;
use libp2p_core::multiaddr::Protocol;
use libp2p_core::multihash::Multihash;
@ -43,6 +44,7 @@ pub struct DialOpts {
extend_addresses_through_behaviour: bool,
role_override: Endpoint,
dial_concurrency_factor_override: Option<NonZeroU8>,
connection_id: ConnectionId,
}
impl DialOpts {
@ -83,6 +85,14 @@ impl DialOpts {
self.peer_id
}
/// Get the [`ConnectionId`] of this dial attempt.
///
/// All future events of this dial will be associated with this ID.
/// See [`DialFailure`](crate::DialFailure) and [`ConnectionEstablished`](crate::behaviour::ConnectionEstablished).
pub fn connection_id(&self) -> ConnectionId {
self.connection_id
}
/// Retrieves the [`PeerId`] from the [`DialOpts`] if specified or otherwise tries to parse it
/// from the multihash in the `/p2p` part of the address, if present.
///
@ -207,6 +217,7 @@ impl WithPeerId {
extend_addresses_through_behaviour: true,
role_override: self.role_override,
dial_concurrency_factor_override: self.dial_concurrency_factor_override,
connection_id: ConnectionId::next(),
}
}
}
@ -262,6 +273,7 @@ impl WithPeerIdWithAddresses {
extend_addresses_through_behaviour: self.extend_addresses_through_behaviour,
role_override: self.role_override,
dial_concurrency_factor_override: self.dial_concurrency_factor_override,
connection_id: ConnectionId::next(),
}
}
}
@ -305,6 +317,7 @@ impl WithoutPeerIdWithAddress {
extend_addresses_through_behaviour: false,
role_override: self.role_override,
dial_concurrency_factor_override: None,
connection_id: ConnectionId::next(),
}
}
}

View File

@ -5,7 +5,7 @@ use crate::handler::{
};
use crate::{
ConnectionHandlerEvent, ConnectionHandlerUpgrErr, KeepAlive, SubstreamProtocol,
THandlerOutEvent,
THandlerInEvent, THandlerOutEvent,
};
use libp2p_core::upgrade::DeniedUpgrade;
use libp2p_core::PeerId;
@ -37,7 +37,7 @@ impl NetworkBehaviour for Behaviour {
&mut self,
_: &mut Context<'_>,
_: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>> {
Poll::Pending
}

View File

@ -4,6 +4,7 @@ use crate::handler::{
ConnectionEvent, ConnectionHandlerEvent, FullyNegotiatedInbound, FullyNegotiatedOutbound,
KeepAlive, SubstreamProtocol,
};
use crate::THandlerInEvent;
use crate::THandlerOutEvent;
use libp2p_core::upgrade::DeniedUpgrade;
use libp2p_core::PeerId;
@ -40,7 +41,7 @@ impl NetworkBehaviour for Behaviour {
&mut self,
_: &mut Context<'_>,
_: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>> {
Poll::Pending
}

View File

@ -92,6 +92,7 @@ pub mod derive_prelude {
pub use crate::NetworkBehaviour;
pub use crate::NetworkBehaviourAction;
pub use crate::PollParameters;
pub use crate::THandlerInEvent;
pub use either::Either;
pub use futures::prelude as futures;
pub use libp2p_core::transport::ListenerId;
@ -162,7 +163,7 @@ type THandler<TBehaviour> = <TBehaviour as NetworkBehaviour>::ConnectionHandler;
/// Custom event that can be received by the [`ConnectionHandler`] of the
/// [`NetworkBehaviour`].
type THandlerInEvent<TBehaviour> =
pub type THandlerInEvent<TBehaviour> =
<<THandler<TBehaviour> as IntoConnectionHandler>::Handler as ConnectionHandler>::InEvent;
/// Custom event that can be produced by the [`ConnectionHandler`] of the [`NetworkBehaviour`].
@ -326,12 +327,6 @@ where
/// List of nodes for which we deny any incoming connection.
banned_peers: HashSet<PeerId>,
/// Connections for which we withhold any reporting. These belong to banned peers.
///
/// Note: Connections to a peer that are established at the time of banning that peer
/// are not added here. Instead they are simply closed.
banned_peer_connections: HashSet<ConnectionId>,
/// Pending event to be delivered to connection handlers
/// (or dropped if the peer disconnected) before the `behaviour`
/// can be polled again.
@ -502,19 +497,13 @@ where
/// swarm.dial("/ip6/::1/tcp/12345".parse::<Multiaddr>().unwrap());
/// ```
pub fn dial(&mut self, opts: impl Into<DialOpts>) -> Result<(), DialError> {
let handler = self.behaviour.new_handler();
self.dial_with_handler(opts.into(), handler)
}
let dial_opts = opts.into();
fn dial_with_handler(
&mut self,
dial_opts: DialOpts,
handler: <TBehaviour as NetworkBehaviour>::ConnectionHandler,
) -> Result<(), DialError> {
let peer_id = dial_opts
.get_or_parse_peer_id()
.map_err(DialError::InvalidPeerId)?;
let condition = dial_opts.peer_condition();
let connection_id = dial_opts.connection_id();
let should_dial = match (condition, peer_id) {
(PeerCondition::Always, _) => true,
@ -530,8 +519,8 @@ where
self.behaviour
.on_swarm_event(FromSwarm::DialFailure(DialFailure {
peer_id,
handler,
error: &e,
connection_id,
}));
return Err(e);
@ -544,8 +533,8 @@ where
self.behaviour
.on_swarm_event(FromSwarm::DialFailure(DialFailure {
peer_id: Some(peer_id),
handler,
error: &error,
connection_id,
}));
return Err(error);
@ -572,8 +561,8 @@ where
self.behaviour
.on_swarm_event(FromSwarm::DialFailure(DialFailure {
peer_id,
handler,
error: &error,
connection_id,
}));
return Err(error);
};
@ -607,18 +596,18 @@ where
match self.pool.add_outgoing(
dials,
peer_id,
handler,
dial_opts.role_override(),
dial_opts.dial_concurrency_override(),
connection_id,
) {
Ok(_connection_id) => Ok(()),
Err((connection_limit, handler)) => {
Ok(()) => Ok(()),
Err(connection_limit) => {
let error = DialError::ConnectionLimit(connection_limit);
self.behaviour
.on_swarm_event(FromSwarm::DialFailure(DialFailure {
peer_id,
handler,
error: &error,
connection_id,
}));
Err(error)
@ -762,65 +751,68 @@ where
peer_id,
id,
endpoint,
other_established_connection_ids,
connection,
concurrent_dial_errors,
established_in,
} => {
if self.banned_peers.contains(&peer_id) {
// Mark the connection for the banned peer as banned, thus withholding any
// future events from the connection to the behaviour.
self.banned_peer_connections.insert(id);
self.pool.disconnect(peer_id);
self.pool.close_connection(connection);
return Some(SwarmEvent::BannedPeer { peer_id, endpoint });
} else {
let num_established = NonZeroU32::new(
u32::try_from(other_established_connection_ids.len() + 1).unwrap(),
)
.expect("n + 1 is always non-zero; qed");
let non_banned_established = other_established_connection_ids
.into_iter()
.filter(|conn_id| !self.banned_peer_connections.contains(conn_id))
.count();
log::debug!(
"Connection established: {:?} {:?}; Total (peer): {}. Total non-banned (peer): {}",
peer_id,
endpoint,
num_established,
non_banned_established + 1,
);
let failed_addresses = concurrent_dial_errors
.as_ref()
.map(|es| {
es.iter()
.map(|(a, _)| a)
.cloned()
.collect::<Vec<Multiaddr>>()
})
.unwrap_or_default();
self.behaviour
.on_swarm_event(FromSwarm::ConnectionEstablished(
behaviour::ConnectionEstablished {
peer_id,
connection_id: id,
endpoint: &endpoint,
failed_addresses: &failed_addresses,
other_established: non_banned_established,
},
));
return Some(SwarmEvent::ConnectionEstablished {
peer_id,
num_established,
endpoint,
concurrent_dial_errors,
established_in,
});
}
let handler = self
.behaviour
.new_handler()
.into_handler(&peer_id, &endpoint);
let other_established_connection_ids = self
.pool
.iter_established_connections_of_peer(&peer_id)
.collect::<Vec<_>>();
let num_established = NonZeroU32::new(
u32::try_from(other_established_connection_ids.len() + 1).unwrap(),
)
.expect("n + 1 is always non-zero; qed");
self.pool
.spawn_connection(id, peer_id, &endpoint, connection, handler);
log::debug!(
"Connection established: {:?} {:?}; Total (peer): {}.",
peer_id,
endpoint,
num_established,
);
let failed_addresses = concurrent_dial_errors
.as_ref()
.map(|es| {
es.iter()
.map(|(a, _)| a)
.cloned()
.collect::<Vec<Multiaddr>>()
})
.unwrap_or_default();
self.behaviour
.on_swarm_event(FromSwarm::ConnectionEstablished(
behaviour::ConnectionEstablished {
peer_id,
connection_id: id,
endpoint: &endpoint,
failed_addresses: &failed_addresses,
other_established: other_established_connection_ids.len(),
},
));
return Some(SwarmEvent::ConnectionEstablished {
peer_id,
num_established,
endpoint,
concurrent_dial_errors,
established_in,
});
}
PoolEvent::PendingOutboundConnectionError {
id: _,
id: connection_id,
error,
handler,
peer,
} => {
let error = error.into();
@ -828,8 +820,8 @@ where
self.behaviour
.on_swarm_event(FromSwarm::DialFailure(DialFailure {
peer_id: peer,
handler,
error: &error,
connection_id,
}));
if let Some(peer) = peer {
@ -844,11 +836,10 @@ where
});
}
PoolEvent::PendingInboundConnectionError {
id: _,
id,
send_back_addr,
local_addr,
error,
handler,
} => {
let error = error.into();
@ -858,7 +849,7 @@ where
local_addr: &local_addr,
send_back_addr: &send_back_addr,
error: &error,
handler,
connection_id: id,
}));
return Some(SwarmEvent::IncomingConnectionError {
local_addr,
@ -892,21 +883,15 @@ where
let endpoint = connected.endpoint;
let num_established =
u32::try_from(remaining_established_connection_ids.len()).unwrap();
let conn_was_reported = !self.banned_peer_connections.remove(&id);
if conn_was_reported {
let remaining_non_banned = remaining_established_connection_ids
.into_iter()
.filter(|conn_id| !self.banned_peer_connections.contains(conn_id))
.count();
self.behaviour
.on_swarm_event(FromSwarm::ConnectionClosed(ConnectionClosed {
peer_id,
connection_id: id,
endpoint: &endpoint,
handler,
remaining_established: remaining_non_banned,
}));
}
self.behaviour
.on_swarm_event(FromSwarm::ConnectionClosed(ConnectionClosed {
peer_id,
connection_id: id,
endpoint: &endpoint,
handler,
remaining_established: num_established as usize,
}));
return Some(SwarmEvent::ConnectionClosed {
peer_id,
endpoint,
@ -915,12 +900,8 @@ where
});
}
PoolEvent::ConnectionEvent { peer_id, id, event } => {
if self.banned_peer_connections.contains(&id) {
log::debug!("Ignoring event from banned peer: {} {:?}.", peer_id, id);
} else {
self.behaviour
.on_connection_handler_event(peer_id, id, event);
}
self.behaviour
.on_connection_handler_event(peer_id, id, event);
}
PoolEvent::AddressChange {
peer_id,
@ -928,15 +909,13 @@ where
new_endpoint,
old_endpoint,
} => {
if !self.banned_peer_connections.contains(&id) {
self.behaviour
.on_swarm_event(FromSwarm::AddressChange(AddressChange {
peer_id,
connection_id: id,
old: &old_endpoint,
new: &new_endpoint,
}));
}
self.behaviour
.on_swarm_event(FromSwarm::AddressChange(AddressChange {
peer_id,
connection_id: id,
old: &old_endpoint,
new: &new_endpoint,
}));
}
}
@ -957,30 +936,30 @@ where
local_addr,
send_back_addr,
} => {
let handler = self.behaviour.new_handler();
let connection_id = ConnectionId::next();
match self.pool.add_incoming(
upgrade,
handler,
IncomingInfo {
local_addr: &local_addr,
send_back_addr: &send_back_addr,
},
connection_id,
) {
Ok(_connection_id) => {
Ok(()) => {
return Some(SwarmEvent::IncomingConnection {
local_addr,
send_back_addr,
});
}
Err((connection_limit, handler)) => {
Err(connection_limit) => {
let error = ListenError::ConnectionLimit(connection_limit);
self.behaviour
.on_swarm_event(FromSwarm::ListenFailure(ListenFailure {
local_addr: &local_addr,
send_back_addr: &send_back_addr,
error: &error,
handler,
connection_id,
}));
log::warn!("Incoming connection rejected: {:?}", connection_limit);
}
@ -1063,15 +1042,15 @@ where
fn handle_behaviour_event(
&mut self,
event: NetworkBehaviourAction<TBehaviour::OutEvent, TBehaviour::ConnectionHandler>,
event: NetworkBehaviourAction<TBehaviour::OutEvent, THandlerInEvent<TBehaviour>>,
) -> Option<SwarmEvent<TBehaviour::OutEvent, THandlerErr<TBehaviour>>> {
match event {
NetworkBehaviourAction::GenerateEvent(event) => {
return Some(SwarmEvent::Behaviour(event))
}
NetworkBehaviourAction::Dial { opts, handler } => {
NetworkBehaviourAction::Dial { opts } => {
let peer_id = opts.get_or_parse_peer_id();
if let Ok(()) = self.dial_with_handler(opts, handler) {
if let Ok(()) = self.dial(opts) {
if let Ok(Some(peer_id)) = peer_id {
return Some(SwarmEvent::Dialing(peer_id));
}
@ -1571,7 +1550,6 @@ where
listened_addrs: HashMap::new(),
external_addrs: Addresses::default(),
banned_peers: HashSet::new(),
banned_peer_connections: HashSet::new(),
pending_event: None,
}
}
@ -1972,24 +1950,22 @@ mod tests {
{
// Setup to test that new connections of banned peers are not reported.
swarm1.dial(addr2.clone()).unwrap();
s1_expected_conns += 1;
stage = Stage::BannedDial;
}
}
Stage::BannedDial => {
if swarm2.network_info().num_peers() == 1 {
// The banned connection was established. Check that it was not reported to
// the behaviour of the banning swarm.
assert_eq!(
swarm2.behaviour.on_connection_established.len(), s2_expected_conns,
"No additional closed connections should be reported for the banned peer"
);
// The banned connection was established. Check that it was not reported to
// the behaviour of the banning swarm.
assert_eq!(
swarm2.behaviour.on_connection_established.len(),
s2_expected_conns,
"No additional closed connections should be reported for the banned peer"
);
// Setup to test that the banned connection is not reported upon closing
// even if the peer is unbanned.
swarm2.unban_peer_id(swarm1_id);
stage = Stage::Unbanned;
}
// Setup to test that the banned connection is not reported upon closing
// even if the peer is unbanned.
swarm2.unban_peer_id(swarm1_id);
stage = Stage::Unbanned;
}
Stage::Unbanned => {
if swarm2.network_info().num_peers() == 0 {
@ -1998,7 +1974,6 @@ mod tests {
swarm2.behaviour.on_connection_closed.len(), s2_expected_conns,
"No additional closed connections should be reported for the banned peer"
);
assert!(swarm2.banned_peer_connections.is_empty());
// Setup to test that a ban lifted does not affect future connections.
for _ in 0..num_connections {

View File

@ -24,7 +24,7 @@ use crate::behaviour::{
};
use crate::{
ConnectionHandler, ConnectionId, IntoConnectionHandler, NetworkBehaviour,
NetworkBehaviourAction, PollParameters, THandlerOutEvent,
NetworkBehaviourAction, PollParameters, THandlerInEvent, THandlerOutEvent,
};
use libp2p_core::{multiaddr::Multiaddr, transport::ListenerId, ConnectedPoint, PeerId};
use std::collections::HashMap;
@ -45,7 +45,7 @@ where
/// The next action to return from `poll`.
///
/// An action is only returned once.
pub next_action: Option<NetworkBehaviourAction<TOutEvent, THandler>>,
pub next_action: Option<NetworkBehaviourAction<TOutEvent, THandler::InEvent>>,
}
impl<THandler, TOutEvent> MockBehaviour<THandler, TOutEvent>
@ -82,7 +82,7 @@ where
&mut self,
_: &mut Context,
_: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>> {
self.next_action.take().map_or(Poll::Pending, Poll::Ready)
}
@ -387,14 +387,14 @@ where
}
FromSwarm::DialFailure(DialFailure {
peer_id,
handler,
connection_id,
error,
}) => {
self.on_dial_failure.push(peer_id);
self.inner
.on_swarm_event(FromSwarm::DialFailure(DialFailure {
peer_id,
handler,
connection_id,
error,
}));
}
@ -478,7 +478,7 @@ where
&mut self,
cx: &mut Context,
args: &mut impl PollParameters,
) -> Poll<NetworkBehaviourAction<Self::OutEvent, Self::ConnectionHandler>> {
) -> Poll<NetworkBehaviourAction<Self::OutEvent, THandlerInEvent<Self>>> {
self.poll += 1;
self.inner.poll(cx, args)
}