diff --git a/Cargo.toml b/Cargo.toml index 516ab03c..2fc3facc 100644 --- a/Cargo.toml +++ b/Cargo.toml @@ -58,21 +58,22 @@ members = [ "misc/multistream-select", "misc/peer-id-generator", "misc/rw-stream-sink", - "transports/dns", + "muxers/mplex", + "muxers/yamux", "protocols/floodsub", "protocols/identify", "protocols/kad", - "protocols/ping", "protocols/observed", - "transports/relay", + "protocols/ping", + "protocols/plaintext", "protocols/secio", - "muxers/mplex", - "muxers/yamux", - "stores/peerstore", "stores/datastore", - "transports/tcp", - "transports/uds", - "transports/websocket", - "transports/timeout", + "stores/peerstore", + "transports/dns", "transports/ratelimit", + "transports/relay", + "transports/tcp", + "transports/timeout", + "transports/uds", + "transports/websocket" ] diff --git a/core/src/either.rs b/core/src/either.rs index 0566dbd3..17827062 100644 --- a/core/src/either.rs +++ b/core/src/either.rs @@ -18,11 +18,42 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::{muxing::{Shutdown, StreamMuxer}, Multiaddr}; use futures::prelude::*; -use muxing::{Shutdown, StreamMuxer}; -use std::io::{Error as IoError, Read, Write}; +use std::{fmt, io::{Error as IoError, Read, Write}}; use tokio_io::{AsyncRead, AsyncWrite}; -use Multiaddr; + +#[derive(Debug, Copy, Clone)] +pub enum EitherError { + A(A), + B(B) +} + +impl fmt::Display for EitherError +where + A: fmt::Display, + B: fmt::Display +{ + fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result { + match self { + EitherError::A(a) => a.fmt(f), + EitherError::B(b) => b.fmt(f) + } + } +} + +impl std::error::Error for EitherError +where + A: fmt::Debug + std::error::Error, + B: fmt::Debug + std::error::Error +{ + fn cause(&self) -> Option<&dyn std::error::Error> { + match self { + EitherError::A(a) => a.cause(), + EitherError::B(b) => b.cause() + } + } +} /// Implements `AsyncRead` and `AsyncWrite` and dispatches all method calls to /// either `First` or `Second`. diff --git a/core/src/lib.rs b/core/src/lib.rs index 11d279b4..dff458ab 100644 --- a/core/src/lib.rs +++ b/core/src/lib.rs @@ -89,7 +89,7 @@ //! //! # fn main() { //! let tcp_transport = libp2p_tcp_transport::TcpConfig::new(); -//! let upgraded = tcp_transport.with_upgrade(libp2p_core::upgrade::PlainTextConfig); +//! let upgraded = tcp_transport.with_upgrade(libp2p_core::upgrade::DeniedUpgrade); //! //! // upgraded.dial(...) // automatically applies the plain text protocol on the socket //! # } @@ -132,31 +132,28 @@ //! extern crate tokio; //! //! use futures::{Future, Stream}; -//! use libp2p_ping::protocol::{Ping, PingOutput}; -//! use libp2p_core::Transport; +//! use libp2p_ping::protocol::Ping; +//! use libp2p_core::{Transport, upgrade::apply_outbound}; //! use tokio::runtime::current_thread::Runtime; //! //! # fn main() { -//! let ping_finished_future = libp2p_tcp_transport::TcpConfig::new() -//! // We have a `TcpConfig` struct that implements `Transport`, and apply a `Ping` upgrade on it. -//! .with_upgrade(Ping::default()) +//! let ping_dialer = libp2p_tcp_transport::TcpConfig::new() +//! // We have a `TcpConfig` struct that implements `Dialer`, and apply a `Ping` upgrade on it. +//! .and_then(|socket, _| { +//! apply_outbound(socket, Ping::default()).map_err(|e| e.into_io_error()) +//! }) //! // TODO: right now the only available protocol is ping, but we want to replace it with //! // something that is more simple to use -//! .dial("127.0.0.1:12345".parse::().unwrap()).unwrap_or_else(|_| panic!()) -//! .and_then(|out| { -//! match out { -//! PingOutput::Ponger(processing) => Box::new(processing) as Box>, -//! PingOutput::Pinger(mut pinger) => { -//! pinger.ping(()); -//! let f = pinger.into_future().map(|_| ()).map_err(|(err, _)| err); -//! Box::new(f) as Box> -//! }, -//! } +//! .dial("/ip4/127.0.0.1/tcp/12345".parse::().unwrap()).unwrap_or_else(|_| panic!()) +//! .and_then(|mut pinger| { +//! pinger.ping(()); +//! let f = pinger.into_future().map(|_| ()).map_err(|(e, _)| e); +//! Box::new(f) as Box> //! }); //! //! // Runs until the ping arrives. //! let mut rt = Runtime::new().unwrap(); -//! let _ = rt.block_on(ping_finished_future).unwrap(); +//! let _ = rt.block_on(ping_dialer).unwrap(); //! # } //! ``` //! @@ -221,4 +218,24 @@ pub use self::muxing::StreamMuxer; pub use self::peer_id::PeerId; pub use self::public_key::PublicKey; pub use self::transport::Transport; -pub use self::upgrade::{ConnectionUpgrade, Endpoint}; +pub use self::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo, UpgradeError}; + +#[derive(Debug, Copy, Clone, PartialEq, Eq, Hash)] +pub enum Endpoint { + /// The socket comes from a dialer. + Dialer, + /// The socket comes from a listener. + Listener, +} + +impl std::ops::Not for Endpoint { + type Output = Endpoint; + + fn not(self) -> Self::Output { + match self { + Endpoint::Dialer => Endpoint::Listener, + Endpoint::Listener => Endpoint::Dialer + } + } +} + diff --git a/core/src/nodes/collection.rs b/core/src/nodes/collection.rs index aadd25a4..6ba78435 100644 --- a/core/src/nodes/collection.rs +++ b/core/src/nodes/collection.rs @@ -18,15 +18,19 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::{ + PeerId, + muxing::StreamMuxer, + nodes::{ + node::Substream, + handled_node_tasks::{HandledNodesEvent, HandledNodesTasks}, + handled_node_tasks::{Task as HandledNodesTask, TaskId}, + handled_node::NodeHandler + } +}; use fnv::FnvHashMap; use futures::prelude::*; -use muxing::StreamMuxer; -use nodes::node::Substream; -use nodes::handled_node_tasks::{HandledNodesEvent, HandledNodesTasks}; -use nodes::handled_node_tasks::{Task as HandledNodesTask, TaskId}; -use nodes::handled_node::NodeHandler; use std::{collections::hash_map::Entry, fmt, io, mem}; -use PeerId; // TODO: make generic over PeerId @@ -276,7 +280,8 @@ impl CollectionStream(&mut self, future: TFut, handler: THandler) -> ReachAttemptId where - TFut: Future + Send + 'static, + TFut: Future + Send + 'static, + TFut::Error: std::error::Error + Send + Sync + 'static, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent> + Send + 'static, TInEvent: Send + 'static, TOutEvent: Send + 'static, diff --git a/core/src/nodes/handled_node_tasks.rs b/core/src/nodes/handled_node_tasks.rs index a33ce3fd..44dc8adf 100644 --- a/core/src/nodes/handled_node_tasks.rs +++ b/core/src/nodes/handled_node_tasks.rs @@ -18,18 +18,25 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::{ + PeerId, + muxing::StreamMuxer, + nodes::{ + handled_node::{HandledNode, NodeHandler}, + node::Substream + } +}; use fnv::FnvHashMap; use futures::{prelude::*, stream, sync::mpsc}; -use muxing::StreamMuxer; -use nodes::node::Substream; -use nodes::handled_node::{HandledNode, NodeHandler}; use smallvec::SmallVec; -use std::collections::hash_map::{Entry, OccupiedEntry}; -use std::io::Error as IoError; -use std::{fmt, mem}; +use std::{ + collections::hash_map::{Entry, OccupiedEntry}, + fmt, + io::{self, Error as IoError}, + mem +}; use tokio_executor; use void::Void; -use PeerId; // TODO: make generic over PeerId @@ -135,10 +142,10 @@ impl HandledNodesTasks(&mut self, future: TFut, handler: THandler) - -> TaskId + pub fn add_reach_attempt(&mut self, future: TFut, handler: THandler) -> TaskId where - TFut: Future + Send + 'static, + TFut: Future + Send + 'static, + TFut::Error: std::error::Error + Send + Sync + 'static, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent> + Send + 'static, TInEvent: Send + 'static, TOutEvent: Send + 'static, @@ -349,7 +356,8 @@ impl Future for NodeTask where TMuxer: StreamMuxer, - TFut: Future, + TFut: Future, + TFut::Error: std::error::Error + Send + Sync + 'static, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent>, { type Item = (); @@ -388,7 +396,8 @@ where }, Err(err) => { // End the task - let event = InToExtMessage::TaskClosed(Err(err), Some(handler)); + let ioerr = IoError::new(io::ErrorKind::Other, err); + let event = InToExtMessage::TaskClosed(Err(ioerr), Some(handler)); let _ = self.events_tx.unbounded_send((event, self.id)); return Ok(Async::Ready(())); } @@ -464,6 +473,7 @@ mod tests { use tests::dummy_muxer::{DummyMuxer, DummyConnectionState}; use tokio::runtime::Builder; use tokio::runtime::current_thread::Runtime; + use void::Void; use {PeerId, PublicKey}; type TestNodeTask = NodeTask< @@ -570,7 +580,7 @@ mod tests { let peer_id = PublicKey::Rsa((0 .. 2048).map(|_| -> u8 { random() }).collect()).into_peer_id(); let mut task_ids = Vec::new(); for _i in 0..self.task_count { - let fut = future::ok((peer_id.clone(), self.muxer.clone())); + let fut = future::ok::<_, Void>((peer_id.clone(), self.muxer.clone())); task_ids.push( handled_nodes.add_reach_attempt(fut, self.handler.clone()) ); @@ -719,7 +729,7 @@ mod tests { assert_eq!(handled_nodes.tasks().count(), 0); assert_eq!(handled_nodes.to_spawn.len(), 0); - handled_nodes.add_reach_attempt( future::empty(), Handler::default() ); + handled_nodes.add_reach_attempt( future::empty::<_, Void>(), Handler::default() ); assert_eq!(handled_nodes.tasks().count(), 1); assert_eq!(handled_nodes.to_spawn.len(), 1); diff --git a/core/src/nodes/protocols_handler.rs b/core/src/nodes/protocols_handler.rs index 33581245..c5c6e0d7 100644 --- a/core/src/nodes/protocols_handler.rs +++ b/core/src/nodes/protocols_handler.rs @@ -18,15 +18,26 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. -use either::EitherOutput; +use crate::{ + either::{EitherError, EitherOutput}, + nodes::handled_node::{NodeHandler, NodeHandlerEndpoint, NodeHandlerEvent}, + upgrade::{ + self, + InboundUpgrade, + InboundUpgradeExt, + OutboundUpgrade, + OutboundUpgradeExt, + UpgradeInfo, + InboundUpgradeApply, + OutboundUpgradeApply, + DeniedUpgrade + } +}; use futures::prelude::*; -use nodes::handled_node::{NodeHandler, NodeHandlerEndpoint, NodeHandlerEvent}; use std::{io, marker::PhantomData, time::Duration}; use tokio_io::{AsyncRead, AsyncWrite}; use tokio_timer::Timeout; -use upgrade::{self, apply::UpgradeApplyFuture, DeniedConnectionUpgrade}; use void::Void; -use {ConnectionUpgrade, Endpoint}; /// Handler for a set of protocols for a specific connection with a remote. /// @@ -81,7 +92,9 @@ pub trait ProtocolsHandler { /// The type of the substream that contains the raw data. type Substream: AsyncRead + AsyncWrite; /// The upgrade for the protocol or protocols handled by this handler. - type Protocol: ConnectionUpgrade; + type InboundProtocol: InboundUpgrade; + /// The upgrade for the protocol or protocols handled by this handler. + type OutboundProtocol: OutboundUpgrade; /// Information about a substream. Can be sent to the handler through a `NodeHandlerEndpoint`, /// and will be passed back in `inject_substream` or `inject_outbound_closed`. type OutboundOpenInfo; @@ -92,15 +105,22 @@ pub trait ProtocolsHandler { /// > context you wouldn't accept one in particular (eg. only allow one substream at /// > a time for a given protocol). The reason is that remotes are allowed to put the /// > list of supported protocols in a cache in order to avoid spurious queries. - fn listen_protocol(&self) -> Self::Protocol; + fn listen_protocol(&self) -> Self::InboundProtocol; + + fn dialer_protocol(&self) -> Self::OutboundProtocol; /// Injects a fully-negotiated substream in the handler. /// /// This method is called when a substream has been successfully opened and negotiated. - fn inject_fully_negotiated( + fn inject_fully_negotiated_inbound( &mut self, - protocol: >::Output, - endpoint: NodeHandlerEndpoint, + protocol: >::Output + ); + + fn inject_fully_negotiated_outbound( + &mut self, + protocol: >::Output, + info: Self::OutboundOpenInfo ); /// Injects an event coming from the outside in the handler. @@ -126,12 +146,7 @@ pub trait ProtocolsHandler { /// > **Note**: If this handler is combined with other handlers, as soon as `poll()` returns /// > `Ok(Async::Ready(None))`, all the other handlers will receive a call to /// > `shutdown()` and will eventually be closed and destroyed. - fn poll( - &mut self, - ) -> Poll< - Option>, - io::Error, - >; + fn poll(&mut self) -> Poll>, io::Error>; /// Adds a closure that turns the input event into something else. #[inline] @@ -296,19 +311,32 @@ where type InEvent = Void; type OutEvent = Void; type Substream = TSubstream; - type Protocol = DeniedConnectionUpgrade; + type InboundProtocol = DeniedUpgrade; + type OutboundProtocol = DeniedUpgrade; type OutboundOpenInfo = Void; #[inline] - fn listen_protocol(&self) -> Self::Protocol { - DeniedConnectionUpgrade + fn listen_protocol(&self) -> Self::InboundProtocol { + DeniedUpgrade } #[inline] - fn inject_fully_negotiated( + fn dialer_protocol(&self) -> Self::OutboundProtocol { + DeniedUpgrade + } + + #[inline] + fn inject_fully_negotiated_inbound( &mut self, - _: >::Output, - _: NodeHandlerEndpoint, + _: >::Output + ) { + } + + #[inline] + fn inject_fully_negotiated_outbound( + &mut self, + _: >::Output, + _: Self::OutboundOpenInfo ) { } @@ -330,7 +358,7 @@ where fn poll( &mut self, ) -> Poll< - Option>, + Option>, io::Error, > { if self.shutting_down { @@ -356,21 +384,35 @@ where type InEvent = TNewIn; type OutEvent = TProtoHandler::OutEvent; type Substream = TProtoHandler::Substream; - type Protocol = TProtoHandler::Protocol; + type InboundProtocol = TProtoHandler::InboundProtocol; + type OutboundProtocol = TProtoHandler::OutboundProtocol; type OutboundOpenInfo = TProtoHandler::OutboundOpenInfo; #[inline] - fn listen_protocol(&self) -> Self::Protocol { + fn listen_protocol(&self) -> Self::InboundProtocol { self.inner.listen_protocol() } #[inline] - fn inject_fully_negotiated( + fn dialer_protocol(&self) -> Self::OutboundProtocol { + self.inner.dialer_protocol() + } + + #[inline] + fn inject_fully_negotiated_inbound( &mut self, - protocol: >::Output, - endpoint: NodeHandlerEndpoint, + protocol: >::Output ) { - self.inner.inject_fully_negotiated(protocol, endpoint) + self.inner.inject_fully_negotiated_inbound(protocol) + } + + #[inline] + fn inject_fully_negotiated_outbound( + &mut self, + protocol: >::Output, + info: Self::OutboundOpenInfo + ) { + self.inner.inject_fully_negotiated_outbound(protocol, info) } #[inline] @@ -399,7 +441,7 @@ where fn poll( &mut self, ) -> Poll< - Option>, + Option>, io::Error, > { self.inner.poll() @@ -420,21 +462,35 @@ where type InEvent = TProtoHandler::InEvent; type OutEvent = TNewOut; type Substream = TProtoHandler::Substream; - type Protocol = TProtoHandler::Protocol; + type InboundProtocol = TProtoHandler::InboundProtocol; + type OutboundProtocol = TProtoHandler::OutboundProtocol; type OutboundOpenInfo = TProtoHandler::OutboundOpenInfo; #[inline] - fn listen_protocol(&self) -> Self::Protocol { + fn listen_protocol(&self) -> Self::InboundProtocol { self.inner.listen_protocol() } #[inline] - fn inject_fully_negotiated( + fn dialer_protocol(&self) -> Self::OutboundProtocol { + self.inner.dialer_protocol() + } + + #[inline] + fn inject_fully_negotiated_inbound( &mut self, - protocol: >::Output, - endpoint: NodeHandlerEndpoint, + protocol: >::Output ) { - self.inner.inject_fully_negotiated(protocol, endpoint) + self.inner.inject_fully_negotiated_inbound(protocol) + } + + #[inline] + fn inject_fully_negotiated_outbound( + &mut self, + protocol: >::Output, + info: Self::OutboundOpenInfo + ) { + self.inner.inject_fully_negotiated_outbound(protocol, info) } #[inline] @@ -461,7 +517,7 @@ where fn poll( &mut self, ) -> Poll< - Option>, + Option>, io::Error, > { Ok(self.inner.poll()?.map(|ev| { @@ -531,12 +587,12 @@ where handler: TProtoHandler, /// Futures that upgrade incoming substreams. negotiating_in: - Vec>>, + Vec>>, /// Futures that upgrade outgoing substreams. The first element of the tuple is the userdata /// to pass back once successfully opened. negotiating_out: Vec<( TProtoHandler::OutboundOpenInfo, - Timeout>, + Timeout>, )>, /// Timeout for incoming substreams negotiation. in_timeout: Duration, @@ -544,7 +600,7 @@ where out_timeout: Duration, /// For each outbound substream request, how to upgrade it. The first element of the tuple /// is the unique identifier (see `unique_dial_upgrade_id`). - queued_dial_upgrades: Vec<(u64, TProtoHandler::Protocol)>, + queued_dial_upgrades: Vec<(u64, TProtoHandler::OutboundProtocol)>, /// Unique identifier assigned to each queued dial upgrade. unique_dial_upgrade_id: u64, } @@ -552,7 +608,8 @@ where impl NodeHandler for NodeHandlerWrapper where TProtoHandler: ProtocolsHandler, - >::NamesIter: Clone, + ::NamesIter: Clone, + ::Substream>>::Error: std::fmt::Debug { type InEvent = TProtoHandler::InEvent; type OutEvent = TProtoHandler::OutEvent; @@ -569,7 +626,7 @@ where match endpoint { NodeHandlerEndpoint::Listener => { let protocol = self.handler.listen_protocol(); - let upgrade = upgrade::apply(substream, protocol, Endpoint::Listener); + let upgrade = upgrade::apply_inbound(substream, protocol); let with_timeout = Timeout::new(upgrade, self.in_timeout); self.negotiating_in.push(with_timeout); } @@ -587,7 +644,7 @@ where }; let (_, proto_upgrade) = self.queued_dial_upgrades.remove(pos); - let upgrade = upgrade::apply(substream, proto_upgrade, Endpoint::Dialer); + let upgrade = upgrade::apply_outbound(substream, proto_upgrade); let with_timeout = Timeout::new(upgrade, self.out_timeout); self.negotiating_out.push((user_data, with_timeout)); } @@ -630,21 +687,15 @@ where self.handler.shutdown(); } - fn poll( - &mut self, - ) -> Poll>, io::Error> { + fn poll(&mut self) -> Poll>, io::Error> { // Continue negotiation of newly-opened substreams on the listening side. // We remove each element from `negotiating_in` one by one and add them back if not ready. for n in (0..self.negotiating_in.len()).rev() { let mut in_progress = self.negotiating_in.swap_remove(n); match in_progress.poll() { - Ok(Async::Ready(upgrade)) => { - self.handler - .inject_fully_negotiated(upgrade, NodeHandlerEndpoint::Listener); - } - Ok(Async::NotReady) => { - self.negotiating_in.push(in_progress); - } + Ok(Async::Ready(upgrade)) => + self.handler.inject_fully_negotiated_inbound(upgrade), + Ok(Async::NotReady) => self.negotiating_in.push(in_progress), // TODO: return a diagnostic event? Err(_err) => {} } @@ -656,8 +707,7 @@ where let (upgr_info, mut in_progress) = self.negotiating_out.swap_remove(n); match in_progress.poll() { Ok(Async::Ready(upgrade)) => { - let endpoint = NodeHandlerEndpoint::Dialer(upgr_info); - self.handler.inject_fully_negotiated(upgrade, endpoint); + self.handler.inject_fully_negotiated_outbound(upgrade, upgr_info); } Ok(Async::NotReady) => { self.negotiating_out.push((upgr_info, in_progress)); @@ -704,45 +754,138 @@ pub struct ProtocolsHandlerSelect { impl ProtocolsHandler for ProtocolsHandlerSelect -where TProto1: ProtocolsHandler, - TProto2: ProtocolsHandler, - TSubstream: AsyncRead + AsyncWrite, - TProto1::Protocol: ConnectionUpgrade, - TProto2::Protocol: ConnectionUpgrade, +where + TProto1: ProtocolsHandler, + TProto2: ProtocolsHandler, + TSubstream: AsyncRead + AsyncWrite, + TProto1::InboundProtocol: InboundUpgrade, + TProto2::InboundProtocol: InboundUpgrade, + TProto1::OutboundProtocol: OutboundUpgrade, + TProto2::OutboundProtocol: OutboundUpgrade { type InEvent = EitherOutput; type OutEvent = EitherOutput; type Substream = TSubstream; - type Protocol = upgrade::OrUpgrade EitherOutput>>, upgrade::toggleable::Toggleable EitherOutput>>>; + + type InboundProtocol = + upgrade::OrUpgrade< + upgrade::Toggleable< + upgrade::MapUpgradeErr< + upgrade::MapUpgrade< + TProto1::InboundProtocol, + fn(TProto1Out) -> EitherOutput + >, + fn(>::Error) -> + EitherError< + >::Error, + >::Error + > + > + >, + upgrade::Toggleable< + upgrade::MapUpgradeErr< + upgrade::MapUpgrade< + TProto2::InboundProtocol, + fn(TProto2Out) -> EitherOutput + >, + fn(>::Error) -> + EitherError< + >::Error, + >::Error + > + > + > + >; + + type OutboundProtocol = + upgrade::OrUpgrade< + upgrade::Toggleable< + upgrade::MapUpgradeErr< + upgrade::MapUpgrade< + TProto1::OutboundProtocol, + fn(TProto1Out) -> EitherOutput + >, + fn(>::Error) -> + EitherError< + >::Error, + >::Error + > + > + >, + upgrade::Toggleable< + upgrade::MapUpgradeErr< + upgrade::MapUpgrade< + TProto2::OutboundProtocol, + fn(TProto2Out) -> EitherOutput + >, + fn(>::Error) -> + EitherError< + >::Error, + >::Error + > + > + > + >; + type OutboundOpenInfo = EitherOutput; #[inline] - fn listen_protocol(&self) -> Self::Protocol { - let proto1 = upgrade::toggleable(upgrade::map::<_, fn(_) -> _>(self.proto1.listen_protocol(), EitherOutput::First)); - let proto2 = upgrade::toggleable(upgrade::map::<_, fn(_) -> _>(self.proto2.listen_protocol(), EitherOutput::Second)); - upgrade::or(proto1, proto2) + fn listen_protocol(&self) -> Self::InboundProtocol { + let proto1 = self.proto1.listen_protocol() + .map_inbound(EitherOutput::First as fn(TProto1Out) -> EitherOutput) + .map_inbound_err(EitherError::A as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + let proto2 = self.proto2.listen_protocol() + .map_inbound(EitherOutput::Second as fn(TProto2Out) -> EitherOutput) + .map_inbound_err(EitherError::B as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + upgrade::toggleable(proto1).or_inbound(upgrade::toggleable(proto2)) } - fn inject_fully_negotiated(&mut self, protocol: >::Output, endpoint: NodeHandlerEndpoint) { + #[inline] + fn dialer_protocol(&self) -> Self::OutboundProtocol { + let proto1 = self.proto1.dialer_protocol() + .map_outbound(EitherOutput::First as fn(TProto1Out) -> EitherOutput) + .map_outbound_err(EitherError::A as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + let proto2 = self.proto2.dialer_protocol() + .map_outbound(EitherOutput::Second as fn(TProto2Out) -> EitherOutput) + .map_outbound_err(EitherError::B as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + upgrade::toggleable(proto1).or_outbound(upgrade::toggleable(proto2)) + } + + fn inject_fully_negotiated_outbound(&mut self, protocol: >::Output, endpoint: Self::OutboundOpenInfo) { match (protocol, endpoint) { - (EitherOutput::First(protocol), NodeHandlerEndpoint::Dialer(EitherOutput::First(info))) => { - self.proto1.inject_fully_negotiated(protocol, NodeHandlerEndpoint::Dialer(info)); - }, - (EitherOutput::Second(protocol), NodeHandlerEndpoint::Dialer(EitherOutput::Second(info))) => { - self.proto2.inject_fully_negotiated(protocol, NodeHandlerEndpoint::Dialer(info)); - }, - (EitherOutput::First(_), NodeHandlerEndpoint::Dialer(EitherOutput::Second(_))) => { + (EitherOutput::First(protocol), EitherOutput::First(info)) => + self.proto1.inject_fully_negotiated_outbound(protocol, info), + (EitherOutput::Second(protocol), EitherOutput::Second(info)) => + self.proto2.inject_fully_negotiated_outbound(protocol, info), + (EitherOutput::First(_), EitherOutput::Second(_)) => + panic!("wrong API usage: the protocol doesn't match the upgrade info"), + (EitherOutput::Second(_), EitherOutput::First(_)) => panic!("wrong API usage: the protocol doesn't match the upgrade info") - }, - (EitherOutput::Second(_), NodeHandlerEndpoint::Dialer(EitherOutput::First(_))) => { - panic!("wrong API usage: the protocol doesn't match the upgrade info") - }, - (EitherOutput::First(protocol), NodeHandlerEndpoint::Listener) => { - self.proto1.inject_fully_negotiated(protocol, NodeHandlerEndpoint::Listener); - }, - (EitherOutput::Second(protocol), NodeHandlerEndpoint::Listener) => { - self.proto2.inject_fully_negotiated(protocol, NodeHandlerEndpoint::Listener); - }, + } + } + + fn inject_fully_negotiated_inbound(&mut self, protocol: >::Output) { + match protocol { + EitherOutput::First(protocol) => + self.proto1.inject_fully_negotiated_inbound(protocol), + EitherOutput::Second(protocol) => + self.proto2.inject_fully_negotiated_inbound(protocol) } } @@ -774,19 +917,33 @@ where TProto1: ProtocolsHandler, self.proto2.shutdown(); } - fn poll(&mut self) -> Poll>, io::Error> { + fn poll(&mut self) -> Poll>, io::Error> { match self.proto1.poll()? { Async::Ready(Some(ProtocolsHandlerEvent::Custom(event))) => { return Ok(Async::Ready(Some(ProtocolsHandlerEvent::Custom(EitherOutput::First(event))))); }, Async::Ready(Some(ProtocolsHandlerEvent::OutboundSubstreamRequest { upgrade, info})) => { let upgrade = { - let proto1 = upgrade::toggleable(upgrade::map::<_, fn(_) -> _>(upgrade, EitherOutput::First)); - let mut proto2 = upgrade::toggleable(upgrade::map::<_, fn(_) -> _>(self.proto2.listen_protocol(), EitherOutput::Second)); - proto2.disable(); - upgrade::or(proto1, proto2) - }; + let proto1 = upgrade + .map_outbound(EitherOutput::First as fn(TProto1Out) -> EitherOutput) + .map_outbound_err(EitherError::A as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + let proto2 = self.proto2.dialer_protocol() + .map_outbound(EitherOutput::Second as fn(TProto2Out) -> EitherOutput) + .map_outbound_err(EitherError::B as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + let proto1 = upgrade::toggleable(proto1); + let mut proto2 = upgrade::toggleable(proto2); + proto2.disable(); + proto1.or_outbound(proto2) + }; return Ok(Async::Ready(Some(ProtocolsHandlerEvent::OutboundSubstreamRequest { upgrade, info: EitherOutput::First(info), @@ -802,12 +959,26 @@ where TProto1: ProtocolsHandler, }, Async::Ready(Some(ProtocolsHandlerEvent::OutboundSubstreamRequest { upgrade, info })) => { let upgrade = { - let mut proto1 = upgrade::toggleable(upgrade::map::<_, fn(_) -> _>(self.proto1.listen_protocol(), EitherOutput::First)); - proto1.disable(); - let proto2 = upgrade::toggleable(upgrade::map::<_, fn(_) -> _>(upgrade, EitherOutput::Second)); - upgrade::or(proto1, proto2) - }; + let proto1 = self.proto1.dialer_protocol() + .map_outbound(EitherOutput::First as fn(TProto1Out) -> EitherOutput) + .map_outbound_err(EitherError::A as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + let proto2 = upgrade + .map_outbound(EitherOutput::Second as fn(TProto2Out) -> EitherOutput) + .map_outbound_err(EitherError::B as fn(>::Error) -> + EitherError< + >::Error, + >::Error + >); + let mut proto1 = upgrade::toggleable(proto1); + proto1.disable(); + let proto2 = upgrade::toggleable(proto2); + proto1.or_outbound(proto2) + }; return Ok(Async::Ready(Some(ProtocolsHandlerEvent::OutboundSubstreamRequest { upgrade, info: EitherOutput::Second(info), diff --git a/core/src/nodes/raw_swarm.rs b/core/src/nodes/raw_swarm.rs index f2fb5148..8d02a9ec 100644 --- a/core/src/nodes/raw_swarm.rs +++ b/core/src/nodes/raw_swarm.rs @@ -18,18 +18,30 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::muxing::StreamMuxer; +use crate::{ + Endpoint, Multiaddr, PeerId, + nodes::{ + collection::{ + CollectionEvent, + CollectionNodeAccept, + CollectionReachEvent, + CollectionStream, + PeerMut as CollecPeerMut, + ReachAttemptId + }, + handled_node::NodeHandler, + node::Substream + }, + nodes::listeners::{ListenersEvent, ListenersStream}, + transport::Transport +}; use fnv::FnvHashMap; use futures::{prelude::*, future}; -use muxing::StreamMuxer; -use nodes::collection::{ - CollectionEvent, CollectionNodeAccept, CollectionReachEvent, CollectionStream, PeerMut as CollecPeerMut, ReachAttemptId, +use std::{ + collections::hash_map::{Entry, OccupiedEntry}, + io::{Error as IoError, ErrorKind as IoErrorKind} }; -use nodes::handled_node::NodeHandler; -use nodes::listeners::{ListenersEvent, ListenersStream}; -use nodes::node::Substream; -use std::collections::hash_map::{Entry, OccupiedEntry}; -use std::io::{Error as IoError, ErrorKind as IoErrorKind}; -use {Endpoint, Multiaddr, PeerId, Transport}; /// Implementation of `Stream` that handles the nodes. pub struct RawSwarm @@ -198,8 +210,7 @@ where TTrans: Transport impl<'a, TTrans, TInEvent, TOutEvent, TMuxer, THandler> IncomingConnectionEvent<'a, TTrans, TInEvent, TOutEvent, THandler> where - TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, + TTrans: Transport, TTrans::ListenerUpgrade: Send + 'static, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent> + Send + 'static, THandler::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary @@ -318,7 +329,7 @@ impl ConnectedPoint { impl RawSwarm where - TTrans: Transport + Clone, + TTrans: Transport + Clone, TMuxer: StreamMuxer, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent> + Send + 'static, THandler::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary @@ -380,7 +391,7 @@ where /// The second parameter is the handler to use if we manage to reach a node. pub fn dial(&mut self, addr: Multiaddr, handler: THandler) -> Result<(), Multiaddr> where - TTrans: Transport + Clone, + TTrans: Transport, TTrans::Dial: Send + 'static, TMuxer: StreamMuxer + Send + Sync + 'static, TMuxer::OutboundSubstream: Send, @@ -463,10 +474,9 @@ where /// /// It is a logic error to call this method if we already have an outgoing attempt to the /// given peer. - fn start_dial_out(&mut self, peer_id: PeerId, handler: THandler, first: Multiaddr, - rest: Vec) + fn start_dial_out(&mut self, peer_id: PeerId, handler: THandler, first: Multiaddr, rest: Vec) where - TTrans: Transport + Clone, + TTrans: Transport, TTrans::Dial: Send + 'static, TMuxer: StreamMuxer + Send + Sync + 'static, TMuxer::OutboundSubstream: Send, @@ -510,7 +520,7 @@ where /// Provides an API similar to `Stream`, except that it cannot error. pub fn poll(&mut self) -> Async> where - TTrans: Transport + Clone, + TTrans: Transport, TTrans::Dial: Send + 'static, TTrans::ListenerUpgrade: Send + 'static, TMuxer: StreamMuxer + Send + Sync + 'static, @@ -524,11 +534,7 @@ where // Start by polling the listeners for events. match self.listeners.poll() { Async::NotReady => (), - Async::Ready(ListenersEvent::Incoming { - upgrade, - listen_addr, - send_back_addr, - }) => { + Async::Ready(ListenersEvent::Incoming { upgrade, listen_addr, send_back_addr }) => { let event = IncomingConnectionEvent { upgrade, listen_addr, @@ -536,14 +542,9 @@ where active_nodes: &mut self.active_nodes, other_reach_attempts: &mut self.reach_attempts.other_reach_attempts, }; - return Async::Ready(RawSwarmEvent::IncomingConnection(event)); } - Async::Ready(ListenersEvent::Closed { - listen_addr, - listener, - result, - }) => { + Async::Ready(ListenersEvent::Closed { listen_addr, listener, result }) => { return Async::Ready(RawSwarmEvent::ListenerClosed { listen_addr, listener, @@ -654,7 +655,6 @@ fn handle_node_reached<'a, TTrans, TMuxer, TInEvent, TOutEvent, THandler>( ) -> (ActionItem, RawSwarmEvent<'a, TTrans, TInEvent, TOutEvent, THandler>) where TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, TMuxer: StreamMuxer + Send + Sync + 'static, TMuxer::OutboundSubstream: Send, TMuxer::Substream: Send, @@ -848,10 +848,15 @@ where impl<'a, TTrans, TMuxer, TInEvent, TOutEvent, THandler> Peer<'a, TTrans, TInEvent, TOutEvent, THandler> where - TTrans: Transport, - TMuxer: StreamMuxer, + TTrans: Transport + Clone, + TTrans::Dial: Send + 'static, + TMuxer: StreamMuxer + Send + Sync + 'static, + TMuxer::OutboundSubstream: Send, + TMuxer::Substream: Send, + TInEvent: Send + 'static, + TOutEvent: Send + 'static, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent> + Send + 'static, - THandler::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary + THandler::OutboundOpenInfo: Send + 'static, { /// If we are connected, returns the `PeerConnected`. #[inline] @@ -889,19 +894,8 @@ where /// > that we are expecting, in which case the handler will be used for this "wrong" /// > node. #[inline] - pub fn or_connect( - self, - addr: Multiaddr, - handler: THandler, - ) -> Result, Self> - where - TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, - TMuxer: StreamMuxer + Send + Sync + 'static, - TMuxer::OutboundSubstream: Send, - TMuxer::Substream: Send, - TInEvent: Send + 'static, - TOutEvent: Send + 'static, + pub fn or_connect(self, addr: Multiaddr, handler: THandler) + -> Result, Self> { self.or_connect_with(move |_| addr, handler) } @@ -916,20 +910,10 @@ where /// > that we are expecting, in which case the handler will be used for this "wrong" /// > node. #[inline] - pub fn or_connect_with( - self, - addr: TFn, - handler: THandler, - ) -> Result, Self> + pub fn or_connect_with(self, addr: TFn, handler: THandler) + -> Result, Self> where TFn: FnOnce(&PeerId) -> Multiaddr, - TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, - TMuxer: StreamMuxer + Send + Sync + 'static, - TMuxer::OutboundSubstream: Send, - TMuxer::Substream: Send, - TInEvent: Send + 'static, - TOutEvent: Send + 'static, { match self { Peer::Connected(peer) => Ok(PeerPotentialConnect::Connected(peer)), @@ -1013,7 +997,7 @@ impl<'a, TInEvent> PeerConnected<'a, TInEvent> { .expect("We insert into connected_points whenever a connection is opened and remove \ only when a connection is closed; the underlying API is guaranteed to always \ deliver a connection closed message after it has been opened, and no two \ - closed messages; qed") + closed messages; qed") } /// Sends an event to the node. @@ -1082,26 +1066,22 @@ where impl<'a, TTrans, TInEvent, TOutEvent, TMuxer, THandler> PeerNotConnected<'a, TTrans, TInEvent, TOutEvent, THandler> where - TTrans: Transport, - TMuxer: StreamMuxer, + TTrans: Transport + Clone, + TTrans::Dial: Send + 'static, + TMuxer: StreamMuxer + Send + Sync + 'static, + TMuxer::OutboundSubstream: Send, + TMuxer::Substream: Send, THandler: NodeHandler, InEvent = TInEvent, OutEvent = TOutEvent> + Send + 'static, THandler::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary + TInEvent: Send + 'static, + TOutEvent: Send + 'static, { /// Attempts a new connection to this node using the given multiaddress. /// /// If we reach a peer but the `PeerId` doesn't correspond to the one we're expecting, then /// the whole connection is immediately closed. #[inline] - pub fn connect(self, addr: Multiaddr, handler: THandler) -> Result, Self> - where - TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, - TMuxer: StreamMuxer + Send + Sync + 'static, - TMuxer::OutboundSubstream: Send, - TMuxer::Substream: Send, - TInEvent: Send + 'static, - TOutEvent: Send + 'static, - { + pub fn connect(self, addr: Multiaddr, handler: THandler) -> Result, Self> { self.connect_inner(handler, addr, Vec::new()) } @@ -1114,20 +1094,10 @@ where /// If we reach a peer but the `PeerId` doesn't correspond to the one we're expecting, then /// the whole connection is immediately closed. #[inline] - pub fn connect_iter( - self, - addrs: TIter, - handler: THandler, - ) -> Result, Self> + pub fn connect_iter(self, addrs: TIter, handler: THandler) + -> Result, Self> where TIter: IntoIterator, - TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, - TMuxer: StreamMuxer + Send + Sync + 'static, - TMuxer::OutboundSubstream: Send, - TMuxer::Substream: Send, - TInEvent: Send + 'static, - TOutEvent: Send + 'static, { let mut addrs = addrs.into_iter(); let first = match addrs.next() { @@ -1139,23 +1109,10 @@ where } /// Inner implementation of `connect`. - fn connect_inner( - self, - handler: THandler, - first: Multiaddr, - rest: Vec, - ) -> Result, Self> - where - TTrans: Transport + Clone, - TTrans::Dial: Send + 'static, - TMuxer: StreamMuxer + Send + Sync + 'static, - TMuxer::OutboundSubstream: Send, - TMuxer::Substream: Send, - TInEvent: Send + 'static, - TOutEvent: Send + 'static, + fn connect_inner(self, handler: THandler, first: Multiaddr, rest: Vec) + -> Result, Self> { self.nodes.start_dial_out(self.peer_id.clone(), handler, first, rest); - Ok(PeerPendingConnect { attempt: match self.nodes.reach_attempts.out_reach_attempts.entry(self.peer_id) { Entry::Occupied(e) => e, diff --git a/core/src/nodes/swarm.rs b/core/src/nodes/swarm.rs index 7cc92b5c..cb41a3aa 100644 --- a/core/src/nodes/swarm.rs +++ b/core/src/nodes/swarm.rs @@ -18,15 +18,19 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::{ + Transport, Multiaddr, PeerId, InboundUpgrade, OutboundUpgrade, UpgradeInfo, + muxing::StreamMuxer, + nodes::{ + handled_node::NodeHandler, + node::Substream, + protocols_handler::{NodeHandlerWrapper, ProtocolsHandler}, + raw_swarm::{RawSwarm, RawSwarmEvent, ConnectedPoint} + }, + topology::Topology +}; use futures::prelude::*; -use muxing::StreamMuxer; -use nodes::handled_node::NodeHandler; -use nodes::node::Substream; -use nodes::protocols_handler::{NodeHandlerWrapper, ProtocolsHandler}; -use nodes::raw_swarm::{RawSwarm, RawSwarmEvent, ConnectedPoint}; -use std::{io, ops::{Deref, DerefMut}}; -use topology::Topology; -use {ConnectionUpgrade, Multiaddr, PeerId, Transport}; +use std::{fmt, io, ops::{Deref, DerefMut}}; /// Contains the state of the network, plus the way it should behave. pub struct Swarm @@ -77,17 +81,23 @@ where TBehaviour: NetworkBehavior, ::OutboundSubstream: Send + 'static, ::Substream: Send + 'static, TTransport: Transport + Clone, - TTransport::Dial: Send + 'static, TTransport::Listener: Send + 'static, TTransport::ListenerUpgrade: Send + 'static, + TTransport::Dial: Send + 'static, TBehaviour::ProtocolsHandler: ProtocolsHandler> + Send + 'static, ::InEvent: Send + 'static, ::OutEvent: Send + 'static, - ::Protocol: ConnectionUpgrade> + Send + 'static, - <::Protocol as ConnectionUpgrade>>::Future: Send + 'static, - <::Protocol as ConnectionUpgrade>>::NamesIter: Clone + Send + 'static, - <::Protocol as ConnectionUpgrade>>::UpgradeIdentifier: Send + 'static, ::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary + ::InboundProtocol: InboundUpgrade> + Send + 'static, + <::InboundProtocol as UpgradeInfo>::NamesIter: Clone + Send + 'static, + <::InboundProtocol as UpgradeInfo>::UpgradeId: Send + 'static, + <::InboundProtocol as InboundUpgrade>>::Error: fmt::Debug + Send + 'static, + <::InboundProtocol as InboundUpgrade>>::Future: Send + 'static, + ::OutboundProtocol: OutboundUpgrade> + Send + 'static, + <::OutboundProtocol as UpgradeInfo>::NamesIter: Send + 'static, + <::OutboundProtocol as UpgradeInfo>::UpgradeId: Send + 'static, + <::OutboundProtocol as OutboundUpgrade>>::Future: Send + 'static, + <::OutboundProtocol as OutboundUpgrade>>::Error: fmt::Debug + Send + 'static, as NodeHandler>::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary TTopology: Topology, { @@ -164,17 +174,23 @@ where TBehaviour: NetworkBehavior, ::OutboundSubstream: Send + 'static, ::Substream: Send + 'static, TTransport: Transport + Clone, - TTransport::Dial: Send + 'static, TTransport::Listener: Send + 'static, TTransport::ListenerUpgrade: Send + 'static, + TTransport::Dial: Send + 'static, TBehaviour::ProtocolsHandler: ProtocolsHandler> + Send + 'static, ::InEvent: Send + 'static, ::OutEvent: Send + 'static, - ::Protocol: ConnectionUpgrade> + Send + 'static, - <::Protocol as ConnectionUpgrade>>::Future: Send + 'static, - <::Protocol as ConnectionUpgrade>>::NamesIter: Clone + Send + 'static, - <::Protocol as ConnectionUpgrade>>::UpgradeIdentifier: Send + 'static, ::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary + ::InboundProtocol: InboundUpgrade> + Send + 'static, + <::InboundProtocol as InboundUpgrade>>::Future: Send + 'static, + <::InboundProtocol as InboundUpgrade>>::Error: fmt::Debug + Send + 'static, + <::InboundProtocol as UpgradeInfo>::NamesIter: Clone + Send + 'static, + <::InboundProtocol as UpgradeInfo>::UpgradeId: Send + 'static, + ::OutboundProtocol: OutboundUpgrade> + Send + 'static, + <::OutboundProtocol as OutboundUpgrade>>::Future: Send + 'static, + <::OutboundProtocol as OutboundUpgrade>>::Error: fmt::Debug + Send + 'static, + <::OutboundProtocol as UpgradeInfo>::NamesIter: Clone + Send + 'static, + <::OutboundProtocol as UpgradeInfo>::UpgradeId: Send + 'static, as NodeHandler>::OutboundOpenInfo: Send + 'static, // TODO: shouldn't be necessary TTopology: Topology, { diff --git a/core/src/transport/mod.rs b/core/src/transport/mod.rs index 21c7107c..9890b0d1 100644 --- a/core/src/transport/mod.rs +++ b/core/src/transport/mod.rs @@ -1,4 +1,4 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. +// Copyright 2017-2018 Parity Technologies (UK) Ltd. // // Permission is hereby granted, free of charge, to any person obtaining a // copy of this software and associated documentation files (the "Software"), @@ -29,12 +29,12 @@ //! `UpgradedNode::or_upgrade` methods, you can combine multiple transports and/or upgrades //! together in a complex chain of protocols negotiation. +use crate::{InboundUpgrade, OutboundUpgrade, Endpoint}; use futures::prelude::*; use multiaddr::Multiaddr; use nodes::raw_swarm::ConnectedPoint; use std::io::Error as IoError; use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; pub mod and_then; pub mod boxed; @@ -50,7 +50,7 @@ pub mod upgrade; pub use self::choice::OrTransport; pub use self::denied::DeniedTransport; pub use self::memory::connector; -pub use self::upgrade::UpgradedNode; +pub use self::upgrade::Upgrade; /// A transport is an object that can be used to produce connections by listening or dialing a /// peer. @@ -178,13 +178,14 @@ pub trait Transport { /// > **Note**: The concept of an *upgrade* for example includes middlewares such *secio* /// > (communication encryption), *multiplex*, but also a protocol handler. #[inline] - fn with_upgrade(self, upgrade: U) -> UpgradedNode + fn with_upgrade(self, upgrade: U) -> Upgrade where Self: Sized, Self::Output: AsyncRead + AsyncWrite, - U: ConnectionUpgrade, + U: InboundUpgrade, + U: OutboundUpgrade { - UpgradedNode::new(self, upgrade) + Upgrade::new(self, upgrade) } /// Wraps this transport inside an upgrade. Whenever a connection that uses this transport diff --git a/core/src/transport/upgrade.rs b/core/src/transport/upgrade.rs index e661b535..77dc8956 100644 --- a/core/src/transport/upgrade.rs +++ b/core/src/transport/upgrade.rs @@ -1,4 +1,4 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. +// Copyright 2017-2018 Parity Technologies (UK) Ltd. // // Permission is hereby granted, free of charge, to any person obtaining a // copy of this software and associated documentation files (the "Software"), @@ -20,173 +20,75 @@ use futures::prelude::*; use multiaddr::Multiaddr; -use std::io::Error as IoError; +use crate::{ + transport::Transport, + upgrade::{OutboundUpgrade, InboundUpgrade, UpgradeInfo, apply_inbound, apply_outbound} +}; use tokio_io::{AsyncRead, AsyncWrite}; -use transport::Transport; -use upgrade::{apply, ConnectionUpgrade, Endpoint}; -/// Implements the `Transport` trait. Dials or listens, then upgrades any dialed or received -/// connection. -/// -/// See the `Transport::with_upgrade` method. -#[derive(Debug, Clone)] -pub struct UpgradedNode { - transports: T, - upgrade: C, +#[derive(Debug, Copy, Clone)] +pub struct Upgrade { inner: T, upgrade: U } + +impl Upgrade { + pub fn new(inner: T, upgrade: U) -> Self { + Upgrade { inner, upgrade } + } } -impl UpgradedNode { - pub fn new(transports: T, upgrade: C) -> UpgradedNode { - UpgradedNode { - transports, - upgrade, +impl Transport for Upgrade +where + D: Transport, + D::Dial: Send + 'static, + D::Listener: Send + 'static, + D::ListenerUpgrade: Send + 'static, + D::Output: AsyncRead + AsyncWrite + Send + 'static, + U: InboundUpgrade, + U: OutboundUpgrade + Send + Clone + 'static, + ::NamesIter: Clone + Send, + ::UpgradeId: Send, + >::Future: Send, + >::Future: Send, + E: std::error::Error + Send + Sync + 'static +{ + type Output = O; + type Listener = Box + Send>; + type ListenerUpgrade = Box + Send>; + type Dial = Box + Send>; + + fn dial(self, addr: Multiaddr) -> Result { + let upgrade = self.upgrade; + match self.inner.dial(addr.clone()) { + Ok(outbound) => { + let future = outbound + .and_then(move |x| apply_outbound(x, upgrade).map_err(|e| { + std::io::Error::new(std::io::ErrorKind::Other, e) + })); + Ok(Box::new(future)) + } + Err((dialer, addr)) => Err((Upgrade::new(dialer, upgrade), addr)) } } -} -impl<'a, T, C> UpgradedNode -where - T: Transport + 'a, - T::Dial: Send, - T::Listener: Send, - T::ListenerUpgrade: Send, - T::Output: Send + AsyncRead + AsyncWrite, - C: ConnectionUpgrade + Send + 'a, - C::NamesIter: Send, - C::Future: Send, - C::UpgradeIdentifier: Send, -{ - /// Returns a reference to the inner `Transport`. - #[inline] - pub fn transport(&self) -> &T { - &self.transports - } - - /// Tries to dial on the `Multiaddr` using the transport that was passed to `new`, then upgrade - /// the connection. - /// - /// Note that this does the same as `Transport::dial`, but with less restrictions on the trait - /// requirements. - #[inline] - pub fn dial( - self, - addr: Multiaddr, - ) -> Result + Send + 'a>, (Self, Multiaddr)> - where - C::NamesIter: Clone, // TODO: not elegant - { - let upgrade = self.upgrade; - - let dialed_fut = match self.transports.dial(addr.clone()) { - Ok(f) => f, - Err((trans, addr)) => { - let builder = UpgradedNode { - transports: trans, - upgrade: upgrade, - }; - - return Err((builder, addr)); - } - }; - - let future = dialed_fut - // Try to negotiate the protocol. - .and_then(move |connection| { - apply(connection, upgrade, Endpoint::Dialer) - }); - - Ok(Box::new(future)) - } - - /// Start listening on the multiaddr using the transport that was passed to `new`. - /// Then whenever a connection is opened, it is upgraded. - /// - /// Note that this does the same as `Transport::listen_on`, but with less restrictions on the - /// trait requirements. - #[inline] - pub fn listen_on( - self, - addr: Multiaddr, - ) -> Result< - ( - Box< - Stream< - Item = (Box + Send + 'a>, Multiaddr), - Error = IoError, - > - + Send - + 'a, - >, - Multiaddr, - ), - (Self, Multiaddr), - > - where - C::NamesIter: Clone, // TODO: not elegant - C: Clone, - { - let upgrade = self.upgrade; - - let (listening_stream, new_addr) = match self.transports.listen_on(addr) { - Ok((l, new_addr)) => (l, new_addr), - Err((trans, addr)) => { - let builder = UpgradedNode { - transports: trans, - upgrade: upgrade, - }; - - return Err((builder, addr)); - } - }; - - // Try to negotiate the protocol. - // Note that failing to negotiate a protocol will never produce a future with an error. - // Instead the `stream` will produce `Ok(Err(...))`. - // `stream` can only produce an `Err` if `listening_stream` produces an `Err`. - let stream = listening_stream.map(move |(connection, client_addr)| { - let upgrade = upgrade.clone(); - let connection = connection - // Try to negotiate the protocol. - .and_then(move |connection| { - apply(connection, upgrade, Endpoint::Listener) - }); - - (Box::new(connection) as Box<_>, client_addr) - }); - - Ok((Box::new(stream), new_addr)) - } -} - -impl Transport for UpgradedNode -where - T: Transport + 'static, - T::Dial: Send, - T::Listener: Send, - T::ListenerUpgrade: Send, - T::Output: Send + AsyncRead + AsyncWrite, - C: ConnectionUpgrade + Clone + Send + 'static, - C::NamesIter: Clone + Send, - C::Future: Send, - C::UpgradeIdentifier: Send, -{ - type Output = C::Output; - type Listener = Box + Send>; - type ListenerUpgrade = Box + Send>; - type Dial = Box + Send>; - - #[inline] fn listen_on(self, addr: Multiaddr) -> Result<(Self::Listener, Multiaddr), (Self, Multiaddr)> { - self.listen_on(addr) + let upgrade = self.upgrade; + match self.inner.listen_on(addr) { + Ok((inbound, addr)) => { + let stream = inbound + .map(move |(future, addr)| { + let upgrade = upgrade.clone(); + let future = future + .and_then(move |x| apply_inbound(x, upgrade).map_err(|e| { + std::io::Error::new(std::io::ErrorKind::Other, e) + })); + (Box::new(future) as Box<_>, addr) + }); + Ok((Box::new(stream), addr)) + } + Err((listener, addr)) => Err((Upgrade::new(listener, upgrade), addr)), + } } - #[inline] - fn dial(self, addr: Multiaddr) -> Result { - self.dial(addr) - } - - #[inline] fn nat_traversal(&self, server: &Multiaddr, observed: &Multiaddr) -> Option { - self.transports.nat_traversal(server, observed) + self.inner.nat_traversal(server, observed) } } diff --git a/core/src/upgrade/apply.rs b/core/src/upgrade/apply.rs index 2e82d981..40698220 100644 --- a/core/src/upgrade/apply.rs +++ b/core/src/upgrade/apply.rs @@ -19,49 +19,53 @@ // DEALINGS IN THE SOFTWARE. use bytes::Bytes; -use futures::{prelude::*, future::Either}; +use crate::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeError}; +use futures::prelude::*; use multistream_select::{self, DialerSelectFuture, ListenerSelectFuture}; -use std::{io::{Error as IoError, ErrorKind as IoErrorKind}, mem}; +use std::mem; use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; -/// Applies a connection upgrade on a socket. -/// -/// Returns a `Future` that returns the outcome of the connection upgrade. -#[inline] -pub fn apply(conn: C, upgrade: U, e: Endpoint) -> UpgradeApplyFuture +pub fn apply_inbound(conn: C, up: U) -> InboundUpgradeApply where - U: ConnectionUpgrade, - U::NamesIter: Clone, // TODO: not elegant C: AsyncRead + AsyncWrite, + U: InboundUpgrade, + U::NamesIter: Clone { - UpgradeApplyFuture { - inner: UpgradeApplyState::Init { - future: negotiate(conn, &upgrade, e), - upgrade, - endpoint: e, - } + let iter = ProtocolNames(up.protocol_names()); + let future = multistream_select::listener_select_proto(conn, iter); + InboundUpgradeApply { + inner: InboundUpgradeApplyState::Init { future, upgrade: up } } } -/// Future, returned from `apply` which performs a connection upgrade. -pub struct UpgradeApplyFuture +pub fn apply_outbound(conn: C, up: U) -> OutboundUpgradeApply where - U: ConnectionUpgrade, - C: AsyncRead + AsyncWrite + C: AsyncRead + AsyncWrite, + U: OutboundUpgrade { - inner: UpgradeApplyState + let iter = ProtocolNames(up.protocol_names()); + let future = multistream_select::dialer_select_proto(conn, iter); + OutboundUpgradeApply { + inner: OutboundUpgradeApplyState::Init { future, upgrade: up } + } } -enum UpgradeApplyState +pub struct InboundUpgradeApply where - U: ConnectionUpgrade, - C: AsyncRead + AsyncWrite + C: AsyncRead + AsyncWrite, + U: InboundUpgrade +{ + inner: InboundUpgradeApplyState +} + +enum InboundUpgradeApplyState +where + C: AsyncRead + AsyncWrite, + U: InboundUpgrade { Init { - future: NegotiationFuture, U::UpgradeIdentifier>, - upgrade: U, - endpoint: Endpoint + future: ListenerSelectFuture, U::UpgradeId>, + upgrade: U }, Upgrade { future: U::Future @@ -69,34 +73,34 @@ where Undefined } -impl Future for UpgradeApplyFuture +impl Future for InboundUpgradeApply where - U: ConnectionUpgrade, - U::NamesIter: Clone, - C: AsyncRead + AsyncWrite + C: AsyncRead + AsyncWrite, + U: InboundUpgrade, + U::NamesIter: Clone { type Item = U::Output; - type Error = IoError; + type Error = UpgradeError; fn poll(&mut self) -> Poll { loop { - match mem::replace(&mut self.inner, UpgradeApplyState::Undefined) { - UpgradeApplyState::Init { mut future, upgrade, endpoint } => { + match mem::replace(&mut self.inner, InboundUpgradeApplyState::Undefined) { + InboundUpgradeApplyState::Init { mut future, upgrade } => { let (upgrade_id, connection) = match future.poll()? { Async::Ready(x) => x, Async::NotReady => { - self.inner = UpgradeApplyState::Init { future, upgrade, endpoint }; + self.inner = InboundUpgradeApplyState::Init { future, upgrade }; return Ok(Async::NotReady) } }; - self.inner = UpgradeApplyState::Upgrade { - future: upgrade.upgrade(connection, upgrade_id, endpoint) + self.inner = InboundUpgradeApplyState::Upgrade { + future: upgrade.upgrade_inbound(connection, upgrade_id) }; } - UpgradeApplyState::Upgrade { mut future } => { + InboundUpgradeApplyState::Upgrade { mut future } => { match future.poll() { Ok(Async::NotReady) => { - self.inner = UpgradeApplyState::Upgrade { future }; + self.inner = InboundUpgradeApplyState::Upgrade { future }; return Ok(Async::NotReady) } Ok(Async::Ready(x)) => { @@ -104,68 +108,82 @@ where return Ok(Async::Ready(x)) } Err(e) => { - debug!("Failed to apply negotiated protocol: {:?}", e); - return Err(e) + debug!("Failed to apply negotiated protocol"); + return Err(UpgradeError::Apply(e)) } } } - UpgradeApplyState::Undefined => - panic!("UpgradeApplyState::poll called after completion") + InboundUpgradeApplyState::Undefined => + panic!("InboundUpgradeApplyState::poll called after completion") } } } } - -/// Negotiates a protocol on a stream. -/// -/// Returns a `Future` that returns the negotiated protocol and the stream. -#[inline] -pub fn negotiate( - connection: C, - upgrade: &U, - endpoint: Endpoint, -) -> NegotiationFuture, U::UpgradeIdentifier> +pub struct OutboundUpgradeApply where - U: ConnectionUpgrade, - U::NamesIter: Clone, // TODO: not elegant C: AsyncRead + AsyncWrite, + U: OutboundUpgrade { - debug!("Starting protocol negotiation"); - let iter = ProtocolNames(upgrade.protocol_names()); - NegotiationFuture { - inner: match endpoint { - Endpoint::Listener => Either::A(multistream_select::listener_select_proto(connection, iter)), - Endpoint::Dialer => Either::B(multistream_select::dialer_select_proto(connection, iter)), - } - } + inner: OutboundUpgradeApplyState } -/// Future, returned by `negotiate`, which negotiates a protocol and stream. -pub struct NegotiationFuture { - inner: Either, DialerSelectFuture> -} - -impl Future for NegotiationFuture +enum OutboundUpgradeApplyState where - R: AsyncRead + AsyncWrite, - I: Iterator + Clone, - M: FnMut(&Bytes, &Bytes) -> bool, + C: AsyncRead + AsyncWrite, + U: OutboundUpgrade { - type Item = (P, R); - type Error = IoError; + Init { + future: DialerSelectFuture, U::UpgradeId>, + upgrade: U + }, + Upgrade { + future: U::Future + }, + Undefined +} + +impl Future for OutboundUpgradeApply +where + C: AsyncRead + AsyncWrite, + U: OutboundUpgrade +{ + type Item = U::Output; + type Error = UpgradeError; fn poll(&mut self) -> Poll { - match self.inner.poll() { - Ok(Async::NotReady) => Ok(Async::NotReady), - Ok(Async::Ready(x)) => { - debug!("Successfully negotiated protocol upgrade"); - Ok(Async::Ready(x)) - } - Err(e) => { - let err = IoError::new(IoErrorKind::Other, e); - debug!("Error while negotiated protocol upgrade: {:?}", err); - Err(err) + loop { + match mem::replace(&mut self.inner, OutboundUpgradeApplyState::Undefined) { + OutboundUpgradeApplyState::Init { mut future, upgrade } => { + let (upgrade_id, connection) = match future.poll()? { + Async::Ready(x) => x, + Async::NotReady => { + self.inner = OutboundUpgradeApplyState::Init { future, upgrade }; + return Ok(Async::NotReady) + } + }; + self.inner = OutboundUpgradeApplyState::Upgrade { + future: upgrade.upgrade_outbound(connection, upgrade_id) + }; + } + OutboundUpgradeApplyState::Upgrade { mut future } => { + match future.poll() { + Ok(Async::NotReady) => { + self.inner = OutboundUpgradeApplyState::Upgrade { future }; + return Ok(Async::NotReady) + } + Ok(Async::Ready(x)) => { + debug!("Successfully applied negotiated protocol"); + return Ok(Async::Ready(x)) + } + Err(e) => { + debug!("Failed to apply negotiated protocol"); + return Err(UpgradeError::Apply(e)) + } + } + } + OutboundUpgradeApplyState::Undefined => + panic!("OutboundUpgradeApplyState::poll called after completion") } } } @@ -191,5 +209,3 @@ where self.0.size_hint() } } - - diff --git a/core/src/upgrade/choice.rs b/core/src/upgrade/choice.rs deleted file mode 100644 index 7dfa8151..00000000 --- a/core/src/upgrade/choice.rs +++ /dev/null @@ -1,122 +0,0 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. -// -// Permission is hereby granted, free of charge, to any person obtaining a -// copy of this software and associated documentation files (the "Software"), -// to deal in the Software without restriction, including without limitation -// the rights to use, copy, modify, merge, publish, distribute, sublicense, -// and/or sell copies of the Software, and to permit persons to whom the -// Software is furnished to do so, subject to the following conditions: -// -// The above copyright notice and this permission notice shall be included in -// all copies or substantial portions of the Software. -// -// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS -// OR IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, -// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE -// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER -// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING -// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER -// DEALINGS IN THE SOFTWARE. - -use bytes::Bytes; -use futures::future; -use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; - -/// Builds a new `ConnectionUpgrade` that chooses between `A` and `B`. -/// -/// If both `A` and `B` are supported by the remote, then `A` will be chosen. -// TODO: write a test for this ^ -#[inline] -pub fn or(me: A, other: B) -> OrUpgrade { - OrUpgrade(me, other) -} - -/// See `upgrade::or`. -#[derive(Debug, Copy, Clone)] -pub struct OrUpgrade(A, B); - -impl ConnectionUpgrade for OrUpgrade -where - C: AsyncRead + AsyncWrite, - A: ConnectionUpgrade, - B: ConnectionUpgrade, -{ - type NamesIter = NamesIterChain; - type UpgradeIdentifier = EitherUpgradeIdentifier; - - #[inline] - fn protocol_names(&self) -> Self::NamesIter { - NamesIterChain { - first: self.0.protocol_names(), - second: self.1.protocol_names(), - } - } - - type Output = O; - type Future = future::Either; - - #[inline] - fn upgrade( - self, - socket: C, - id: Self::UpgradeIdentifier, - ty: Endpoint, - ) -> Self::Future { - match id { - EitherUpgradeIdentifier::First(id) => { - future::Either::A(self.0.upgrade(socket, id, ty)) - } - EitherUpgradeIdentifier::Second(id) => { - future::Either::B(self.1.upgrade(socket, id, ty)) - } - } - } -} - -/// Internal struct used by the `OrUpgrade` trait. -#[derive(Debug, Copy, Clone)] -pub enum EitherUpgradeIdentifier { - First(A), - Second(B), -} - -/// Internal type used by the `OrUpgrade` struct. -/// -/// > **Note**: This type is needed because of the lack of `-> impl Trait` in Rust. It can be -/// > removed eventually. -#[derive(Debug, Copy, Clone)] -pub struct NamesIterChain { - first: A, - second: B, -} - -impl Iterator for NamesIterChain -where - A: Iterator, - B: Iterator, -{ - type Item = (Bytes, EitherUpgradeIdentifier); - - #[inline] - fn next(&mut self) -> Option { - if let Some((name, id)) = self.first.next() { - return Some((name, EitherUpgradeIdentifier::First(id))); - } - if let Some((name, id)) = self.second.next() { - return Some((name, EitherUpgradeIdentifier::Second(id))); - } - None - } - - #[inline] - fn size_hint(&self) -> (usize, Option) { - let (min1, max1) = self.first.size_hint(); - let (min2, max2) = self.second.size_hint(); - let max = match (max1, max2) { - (Some(max1), Some(max2)) => max1.checked_add(max2), - _ => None, - }; - (min1.saturating_add(min2), max) - } -} diff --git a/core/src/upgrade/denied.rs b/core/src/upgrade/denied.rs index 1ead49f9..95b152bb 100644 --- a/core/src/upgrade/denied.rs +++ b/core/src/upgrade/denied.rs @@ -1,4 +1,4 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. +// Copyright 2017-2018 Parity Technologies (UK) Ltd. // // Permission is hereby granted, free of charge, to any person obtaining a // copy of this software and associated documentation files (the "Software"), @@ -19,31 +19,40 @@ // DEALINGS IN THE SOFTWARE. use bytes::Bytes; -use futures::prelude::*; -use std::{io, iter}; -use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; +use crate::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; +use futures::future::FutureResult; +use std::iter; +use void::{unreachable, Void}; -/// Implementation of `ConnectionUpgrade` that always fails to negotiate. #[derive(Debug, Copy, Clone)] -pub struct DeniedConnectionUpgrade; +pub struct DeniedUpgrade; -impl ConnectionUpgrade for DeniedConnectionUpgrade -where - C: AsyncRead + AsyncWrite, -{ - type NamesIter = iter::Empty<(Bytes, ())>; - type UpgradeIdentifier = (); // TODO: could use `!` - type Output = (); // TODO: could use `!` - type Future = Box + Send + Sync>; // TODO: could use `!` +impl UpgradeInfo for DeniedUpgrade { + type UpgradeId = Void; + type NamesIter = iter::Empty<(Bytes, Self::UpgradeId)>; - #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::empty() } +} - #[inline] - fn upgrade(self, _: C, _: Self::UpgradeIdentifier, _: Endpoint) -> Self::Future { - unreachable!("the denied connection upgrade always fails to negotiate") +impl InboundUpgrade for DeniedUpgrade { + type Output = Void; + type Error = Void; + type Future = FutureResult; + + fn upgrade_inbound(self, _: C, id: Self::UpgradeId) -> Self::Future { + unreachable(id) } } + +impl OutboundUpgrade for DeniedUpgrade { + type Output = Void; + type Error = Void; + type Future = FutureResult; + + fn upgrade_outbound(self, _: C, id: Self::UpgradeId) -> Self::Future { + unreachable(id) + } +} + diff --git a/core/src/upgrade/error.rs b/core/src/upgrade/error.rs new file mode 100644 index 00000000..6fe7413c --- /dev/null +++ b/core/src/upgrade/error.rs @@ -0,0 +1,92 @@ +// Copyright 2018 Parity Technologies (UK) Ltd. +// +// Permission is hereby granted, free of charge, to any person obtaining a +// copy of this software and associated documentation files (the "Software"), +// to deal in the Software without restriction, including without limitation +// the rights to use, copy, modify, merge, publish, distribute, sublicense, +// and/or sell copies of the Software, and to permit persons to whom the +// Software is furnished to do so, subject to the following conditions: +// +// The above copyright notice and this permission notice shall be included in +// all copies or substantial portions of the Software. +// +// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS +// OR IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, +// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE +// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER +// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING +// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER +// DEALINGS IN THE SOFTWARE. + +use multistream_select::ProtocolChoiceError; +use std::fmt; + +#[derive(Debug)] +pub enum UpgradeError { + Select(ProtocolChoiceError), + Apply(E), + #[doc(hidden)] + __Nonexhaustive +} + +impl UpgradeError +where + E: std::error::Error + Send + Sync + 'static +{ + pub fn into_io_error(self) -> std::io::Error { + std::io::Error::new(std::io::ErrorKind::Other, self) + } +} + +impl UpgradeError { + pub fn map_err(self, f: F) -> UpgradeError + where + F: FnOnce(E) -> T + { + match self { + UpgradeError::Select(e) => UpgradeError::Select(e), + UpgradeError::Apply(e) => UpgradeError::Apply(f(e)), + UpgradeError::__Nonexhaustive => UpgradeError::__Nonexhaustive + } + } + + pub fn from_err(self) -> UpgradeError + where + T: From + { + self.map_err(Into::into) + } +} + +impl fmt::Display for UpgradeError +where + E: fmt::Display +{ + fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result { + match self { + UpgradeError::Select(e) => write!(f, "select error: {}", e), + UpgradeError::Apply(e) => write!(f, "upgrade apply error: {}", e), + UpgradeError::__Nonexhaustive => f.write_str("__Nonexhaustive") + } + } +} + +impl std::error::Error for UpgradeError +where + E: std::error::Error +{ + fn cause(&self) -> Option<&dyn std::error::Error> { + match self { + UpgradeError::Select(e) => Some(e), + UpgradeError::Apply(e) => Some(e), + UpgradeError::__Nonexhaustive => None + } + } +} + +impl From for UpgradeError { + fn from(e: ProtocolChoiceError) -> Self { + UpgradeError::Select(e) + } +} + diff --git a/core/src/upgrade/loop_upg.rs b/core/src/upgrade/loop_upg.rs deleted file mode 100644 index f597c9ec..00000000 --- a/core/src/upgrade/loop_upg.rs +++ /dev/null @@ -1,138 +0,0 @@ -// Copyright 2018 Parity Technologies (UK) Ltd. -// -// Permission is hereby granted, free of charge, to any person obtaining a -// copy of this software and associated documentation files (the "Software"), -// to deal in the Software without restriction, including without limitation -// the rights to use, copy, modify, merge, publish, distribute, sublicense, -// and/or sell copies of the Software, and to permit persons to whom the -// Software is furnished to do so, subject to the following conditions: -// -// The above copyright notice and this permission notice shall be included in -// all copies or substantial portions of the Software. -// -// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS -// OR IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, -// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE -// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER -// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING -// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER -// DEALINGS IN THE SOFTWARE. - -use futures::{future, future::Loop as FutLoop, prelude::*}; -use std::io::{Error as IoError, ErrorKind as IoErrorKind}; -use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{negotiate, ConnectionUpgrade, Endpoint}; - -/// Looping connection upgrade. -/// -/// Applies a modifier around a `ConnectionUpgrade`. -/// The `ConnectionUpgrade` is expected to produce a `Loop`. If upgrading produces -/// `Loop::Continue`, then the protocol will be negotiated again on the returned stream. -/// If upgrading produces `Loop::Break`, then the loop will stop. -/// -/// This is useful for upgrades that produce a stream over which you want to negotiate a protocol. -/// -/// Note that there is a maximum number of looping after which a runtime error is produced, in -/// order to avoid DoS attacks if your code happens to be wrong. -#[inline] -pub fn loop_upg(inner: U) -> LoopUpg { - LoopUpg { inner } -} - -/// Maximum number of loops after which a runtime error is produced. -pub const MAX_LOOPS: u32 = 64; - -/// See the documentation of `loop_upg`. -pub enum Loop { - /// Looping should continue. `Socket` must implement `AsyncRead` and `AsyncWrite`, and will - /// be used to continue negotiating a protocol. `State` is passed around and can contain - /// anything. - Continue(State, Socket), - /// Stop looping. `Final` is the output of the `loop_upg`. - Break(Final), -} - -/// Looping connection upgrade. -/// -/// See the documentation of `loop_upg`. -#[derive(Debug, Copy, Clone)] -pub struct LoopUpg { - inner: Inner, -} - -// TODO: 'static :-/ -impl ConnectionUpgrade<(State, Socket)> - for LoopUpg -where - State: Send + 'static, - Socket: AsyncRead + AsyncWrite + Send + 'static, - Inner: ConnectionUpgrade< - (State, Socket), - Output = Loop, - > + Clone - + Send - + 'static, - Inner::NamesIter: Clone + Send + 'static, - Inner::UpgradeIdentifier: Send, - Inner::Future: Send, - Out: Send + 'static, -{ - type NamesIter = Inner::NamesIter; - type UpgradeIdentifier = Inner::UpgradeIdentifier; - - fn protocol_names(&self) -> Self::NamesIter { - self.inner.protocol_names() - } - - type Output = Out; - type Future = Box + Send>; - - fn upgrade( - self, - (state, socket): (State, Socket), - id: Self::UpgradeIdentifier, - endpoint: Endpoint, - ) -> Self::Future { - let inner = self.inner; - - let fut = future::loop_fn( - (state, socket, id, MAX_LOOPS), - move |(state, socket, id, loops_remaining)| { - // When we enter a recursion of the `loop_fn`, a protocol has already been - // negotiated. So what we have to do is upgrade then negotiate the next protocol - // (if necessary), and then only continue iteration in the `future::loop_fn`. - let inner = inner.clone(); - inner - .clone() - .upgrade((state, socket), id, endpoint) - .and_then(move |loop_out| match loop_out { - Loop::Continue(state, socket) => { - // Produce an error if we reached the recursion limit. - if loops_remaining == 0 { - return future::Either::B(future::err(IoError::new( - IoErrorKind::Other, - "protocol negotiation maximum recursion limit reached", - ))); - } - - let nego = negotiate(socket, &inner, endpoint); - let fut = nego.map(move |(id, socket)| { - FutLoop::Continue(( - state, - socket, - id, - loops_remaining - 1, - )) - }); - future::Either::A(fut) - } - Loop::Break(fin) => { - future::Either::B(future::ok(FutLoop::Break(fin))) - } - }) - }, - ); - - Box::new(fut) as Box<_> - } -} diff --git a/core/src/upgrade/map.rs b/core/src/upgrade/map.rs index f3faad13..7845869b 100644 --- a/core/src/upgrade/map.rs +++ b/core/src/upgrade/map.rs @@ -1,4 +1,4 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. +// Copyright 2018 Parity Technologies (UK) Ltd. // // Permission is hereby granted, free of charge, to any person obtaining a // copy of this software and associated documentation files (the "Software"), @@ -19,47 +19,114 @@ // DEALINGS IN THE SOFTWARE. use futures::prelude::*; -use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; +use crate::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; -/// Applies a closure on the output of a connection upgrade. -#[inline] -pub fn map(upgrade: U, map: F) -> Map { - Map { upgrade, map } +#[derive(Debug, Clone)] +pub struct MapUpgrade { upgrade: U, fun: F } + +impl MapUpgrade { + pub fn new(upgrade: U, fun: F) -> Self { + MapUpgrade { upgrade, fun } + } } -/// Application of a closure on the output of a connection upgrade. -#[derive(Debug, Copy, Clone)] -pub struct Map { - upgrade: U, - map: F, -} - -impl ConnectionUpgrade for Map +impl UpgradeInfo for MapUpgrade where - U: ConnectionUpgrade, - C: AsyncRead + AsyncWrite, - F: FnOnce(U::Output) -> O, + U: UpgradeInfo { + type UpgradeId = U::UpgradeId; type NamesIter = U::NamesIter; - type UpgradeIdentifier = U::UpgradeIdentifier; fn protocol_names(&self) -> Self::NamesIter { self.upgrade.protocol_names() } +} - type Output = O; +impl InboundUpgrade for MapUpgrade +where + U: InboundUpgrade, + F: FnOnce(U::Output) -> T +{ + type Output = T; + type Error = U::Error; type Future = MapFuture; - fn upgrade( - self, - socket: C, - id: Self::UpgradeIdentifier, - ty: Endpoint, - ) -> Self::Future { + fn upgrade_inbound(self, sock: C, id: Self::UpgradeId) -> Self::Future { MapFuture { - inner: self.upgrade.upgrade(socket, id, ty), - map: Some(self.map), + inner: self.upgrade.upgrade_inbound(sock, id), + map: Some(self.fun) + } + } +} + +impl OutboundUpgrade for MapUpgrade +where + U: OutboundUpgrade, + F: FnOnce(U::Output) -> T +{ + type Output = T; + type Error = U::Error; + type Future = MapFuture; + + fn upgrade_outbound(self, sock: C, id: Self::UpgradeId) -> Self::Future { + MapFuture { + inner: self.upgrade.upgrade_outbound(sock, id), + map: Some(self.fun) + } + } +} + +#[derive(Debug, Clone)] +pub struct MapUpgradeErr { upgrade: U, fun: F } + +impl MapUpgradeErr { + pub fn new(upgrade: U, fun: F) -> Self { + MapUpgradeErr { upgrade, fun } + } +} + +impl UpgradeInfo for MapUpgradeErr +where + U: UpgradeInfo +{ + type UpgradeId = U::UpgradeId; + type NamesIter = U::NamesIter; + + fn protocol_names(&self) -> Self::NamesIter { + self.upgrade.protocol_names() + } +} + +impl InboundUpgrade for MapUpgradeErr +where + U: InboundUpgrade, + F: FnOnce(U::Error) -> T +{ + type Output = U::Output; + type Error = T; + type Future = MapErrFuture; + + fn upgrade_inbound(self, sock: C, id: Self::UpgradeId) -> Self::Future { + MapErrFuture { + fut: self.upgrade.upgrade_inbound(sock, id), + fun: Some(self.fun) + } + } +} + +impl OutboundUpgrade for MapUpgradeErr +where + U: OutboundUpgrade, + F: FnOnce(U::Error) -> T, +{ + type Output = U::Output; + type Error = T; + type Future = MapErrFuture; + + fn upgrade_outbound(self, sock: C, id: Self::UpgradeId) -> Self::Future { + MapErrFuture { + fut: self.upgrade.upgrade_outbound(sock, id), + fun: Some(self.fun) } } } @@ -70,8 +137,9 @@ pub struct MapFuture { } impl Future for MapFuture -where TInnerFut: Future, - TMap: FnOnce(TIn) -> TOut, +where + TInnerFut: Future, + TMap: FnOnce(TIn) -> TOut, { type Item = TOut; type Error = TInnerFut::Error; @@ -82,3 +150,29 @@ where TInnerFut: Future, Ok(Async::Ready(map(item))) } } + +pub struct MapErrFuture { + fut: T, + fun: Option, +} + +impl Future for MapErrFuture +where + T: Future, + F: FnOnce(E) -> A, +{ + type Item = T::Item; + type Error = A; + + fn poll(&mut self) -> Poll { + match self.fut.poll() { + Ok(Async::NotReady) => Ok(Async::NotReady), + Ok(Async::Ready(x)) => Ok(Async::Ready(x)), + Err(e) => { + let f = self.fun.take().expect("Future has not resolved yet"); + Err(f(e)) + } + } + } +} + diff --git a/core/src/upgrade/mod.rs b/core/src/upgrade/mod.rs index 279d5a9b..7b7c2e8a 100644 --- a/core/src/upgrade/mod.rs +++ b/core/src/upgrade/mod.rs @@ -1,4 +1,4 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. +// Copyright 2018 Parity Technologies (UK) Ltd. // // Permission is hereby granted, free of charge, to any person obtaining a // copy of this software and associated documentation files (the "Software"), @@ -18,20 +18,101 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. -pub mod apply; -pub mod choice; -pub mod denied; -pub mod loop_upg; -pub mod map; -pub mod plaintext; -pub mod toggleable; -pub mod traits; +mod apply; +mod denied; +mod error; +mod map; +mod or; +mod toggleable; + +use bytes::Bytes; +use futures::future::Future; + +pub use self::{ + apply::{apply_inbound, apply_outbound, InboundUpgradeApply, OutboundUpgradeApply}, + denied::DeniedUpgrade, + error::UpgradeError, + map::{MapUpgrade, MapUpgradeErr}, + or::OrUpgrade, + toggleable::{toggleable, Toggleable} +}; + +pub trait UpgradeInfo { + type UpgradeId; + type NamesIter: Iterator; + + fn protocol_names(&self) -> Self::NamesIter; +} + +pub trait InboundUpgrade: UpgradeInfo { + type Output; + type Error; + type Future: Future; + + fn upgrade_inbound(self, socket: C, id: Self::UpgradeId) -> Self::Future; +} + +pub trait InboundUpgradeExt: InboundUpgrade { + fn map_inbound(self, f: F) -> MapUpgrade + where + Self: Sized, + F: FnOnce(Self::Output) -> T + { + MapUpgrade::new(self, f) + } + + fn map_inbound_err(self, f: F) -> MapUpgradeErr + where + Self: Sized, + F: FnOnce(Self::Error) -> T + { + MapUpgradeErr::new(self, f) + } + + fn or_inbound(self, upgrade: U) -> OrUpgrade + where + Self: Sized, + U: InboundUpgrade + { + OrUpgrade::new(self, upgrade) + } +} + +impl> InboundUpgradeExt for U {} + +pub trait OutboundUpgrade: UpgradeInfo { + type Output; + type Error; + type Future: Future; + + fn upgrade_outbound(self, socket: C, id: Self::UpgradeId) -> Self::Future; +} + +pub trait OutboundUpgradeExt: OutboundUpgrade { + fn map_outbound(self, f: F) -> MapUpgrade + where + Self: Sized, + F: FnOnce(Self::Output) -> T + { + MapUpgrade::new(self, f) + } + + fn map_outbound_err(self, f: F) -> MapUpgradeErr + where + Self: Sized, + F: FnOnce(Self::Error) -> T + { + MapUpgradeErr::new(self, f) + } + + fn or_outbound(self, upgrade: U) -> OrUpgrade + where + Self: Sized, + U: OutboundUpgrade + { + OrUpgrade::new(self, upgrade) + } +} + +impl> OutboundUpgradeExt for U {} -pub use self::apply::{apply, negotiate}; -pub use self::choice::{or, OrUpgrade}; -pub use self::denied::DeniedConnectionUpgrade; -pub use self::loop_upg::{loop_upg, Loop}; -pub use self::map::map; -pub use self::plaintext::PlainTextConfig; -pub use self::toggleable::toggleable; -pub use self::traits::{ConnectionUpgrade, Endpoint}; diff --git a/core/src/upgrade/or.rs b/core/src/upgrade/or.rs new file mode 100644 index 00000000..8be83e15 --- /dev/null +++ b/core/src/upgrade/or.rs @@ -0,0 +1,108 @@ +// Copyright 2018 Parity Technologies (UK) Ltd. +// +// Permission is hereby granted, free of charge, to any person obtaining a +// copy of this software and associated documentation files (the "Software"), +// to deal in the Software without restriction, including without limitation +// the rights to use, copy, modify, merge, publish, distribute, sublicense, +// and/or sell copies of the Software, and to permit persons to whom the +// Software is furnished to do so, subject to the following conditions: +// +// The above copyright notice and this permission notice shall be included in +// all copies or substantial portions of the Software. +// +// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS +// OR IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, +// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE +// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER +// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING +// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER +// DEALINGS IN THE SOFTWARE. + +use bytes::Bytes; +use futures::future::Either; +use crate::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; + +#[derive(Debug, Clone)] +pub struct OrUpgrade(A, B); + +impl OrUpgrade { + pub fn new(a: A, b: B) -> Self { + OrUpgrade(a, b) + } +} + +impl UpgradeInfo for OrUpgrade +where + A: UpgradeInfo, + B: UpgradeInfo +{ + type UpgradeId = Either; + type NamesIter = NamesIterChain; + + fn protocol_names(&self) -> Self::NamesIter { + NamesIterChain(self.0.protocol_names(), self.1.protocol_names()) + } +} + +impl InboundUpgrade for OrUpgrade +where + A: InboundUpgrade, + B: InboundUpgrade, +{ + type Output = T; // TODO: different output types + type Error = E; // TODO: different error types + type Future = Either; + + fn upgrade_inbound(self, sock: C, id: Self::UpgradeId) -> Self::Future { + match id { + Either::A(id) => Either::A(self.0.upgrade_inbound(sock, id)), + Either::B(id) => Either::B(self.1.upgrade_inbound(sock, id)) + } + } +} + +impl OutboundUpgrade for OrUpgrade +where + A: OutboundUpgrade, + B: OutboundUpgrade, +{ + type Output = T; // TODO: different output types + type Error = E; // TODO: different error types + type Future = Either; + + fn upgrade_outbound(self, sock: C, id: Self::UpgradeId) -> Self::Future { + match id { + Either::A(id) => Either::A(self.0.upgrade_outbound(sock, id)), + Either::B(id) => Either::B(self.1.upgrade_outbound(sock, id)) + } + } +} + +#[derive(Debug, Clone)] +pub struct NamesIterChain(A, B); + +impl Iterator for NamesIterChain +where + A: Iterator, + B: Iterator, +{ + type Item = (Bytes, Either); + + fn next(&mut self) -> Option { + if let Some((name, id)) = self.0.next() { + return Some((name, Either::A(id))) + } + if let Some((name, id)) = self.1.next() { + return Some((name, Either::B(id))) + } + None + } + + fn size_hint(&self) -> (usize, Option) { + let (min1, max1) = self.0.size_hint(); + let (min2, max2) = self.1.size_hint(); + let max = max1.and_then(move |m1| max2.and_then(move |m2| m1.checked_add(m2))); + (min1.saturating_add(min2), max) + } +} + diff --git a/core/src/upgrade/toggleable.rs b/core/src/upgrade/toggleable.rs index 5383ca46..e694e26d 100644 --- a/core/src/upgrade/toggleable.rs +++ b/core/src/upgrade/toggleable.rs @@ -18,12 +18,11 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; use futures::future; -use std::io::Error as IoError; -use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; -/// Wraps around a `ConnectionUpgrade` and makes it possible to enable or disable an upgrade. +/// Wraps around a `InboundUpgrade` or `OutboundUpgrade` and makes it possible +/// to enable or disable the upgrade. #[inline] pub fn toggleable(upgrade: U) -> Toggleable { Toggleable { @@ -32,7 +31,7 @@ pub fn toggleable(upgrade: U) -> Toggleable { } } -/// See `upgrade::toggleable`. +/// See `toggleable`. #[derive(Debug, Copy, Clone)] pub struct Toggleable { inner: U, @@ -65,13 +64,12 @@ impl Toggleable { } } -impl ConnectionUpgrade for Toggleable +impl UpgradeInfo for Toggleable where - C: AsyncRead + AsyncWrite, - U: ConnectionUpgrade, + U: UpgradeInfo { + type UpgradeId = U::UpgradeId; type NamesIter = ToggleableIter; - type UpgradeIdentifier = U::UpgradeIdentifier; #[inline] fn protocol_names(&self) -> Self::NamesIter { @@ -80,19 +78,38 @@ where enabled: self.enabled, } } +} +impl InboundUpgrade for Toggleable +where + U: InboundUpgrade +{ type Output = U::Output; - type Future = future::Either, U::Future>; + type Error = U::Error; + type Future = future::Either, U::Future>; #[inline] - fn upgrade( - self, - socket: C, - id: Self::UpgradeIdentifier, - ty: Endpoint, - ) -> Self::Future { + fn upgrade_inbound(self, socket: C, id: Self::UpgradeId) -> Self::Future { if self.enabled { - future::Either::B(self.inner.upgrade(socket, id, ty)) + future::Either::B(self.inner.upgrade_inbound(socket, id)) + } else { + future::Either::A(future::empty()) + } + } +} + +impl OutboundUpgrade for Toggleable +where + U: OutboundUpgrade +{ + type Output = U::Output; + type Error = U::Error; + type Future = future::Either, U::Future>; + + #[inline] + fn upgrade_outbound(self, socket: C, id: Self::UpgradeId) -> Self::Future { + if self.enabled { + future::Either::B(self.inner.upgrade_outbound(socket, id)) } else { future::Either::A(future::empty()) } @@ -130,4 +147,7 @@ where I: Iterator } impl ExactSizeIterator for ToggleableIter -where I: ExactSizeIterator {} +where + I: ExactSizeIterator +{} + diff --git a/core/src/upgrade/traits.rs b/core/src/upgrade/traits.rs deleted file mode 100644 index 7f987b65..00000000 --- a/core/src/upgrade/traits.rs +++ /dev/null @@ -1,79 +0,0 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. -// -// Permission is hereby granted, free of charge, to any person obtaining a -// copy of this software and associated documentation files (the "Software"), -// to deal in the Software without restriction, including without limitation -// the rights to use, copy, modify, merge, publish, distribute, sublicense, -// and/or sell copies of the Software, and to permit persons to whom the -// Software is furnished to do so, subject to the following conditions: -// -// The above copyright notice and this permission notice shall be included in -// all copies or substantial portions of the Software. -// -// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS -// OR IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, -// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE -// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER -// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING -// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER -// DEALINGS IN THE SOFTWARE. - -use bytes::Bytes; -use futures::future::Future; -use std::{io::Error as IoError, ops::Not}; - -/// Type of connection for the upgrade. -#[derive(Debug, Copy, Clone, PartialEq, Eq, Hash)] -pub enum Endpoint { - /// The socket comes from a dialer. - Dialer, - /// The socket comes from a listener. - Listener, -} - -impl Not for Endpoint { - type Output = Endpoint; - - fn not(self) -> Self::Output { - match self { - Endpoint::Dialer => Endpoint::Listener, - Endpoint::Listener => Endpoint::Dialer - } - } -} - - -/// Implemented on structs that describe a possible upgrade to a connection between two peers. -/// -/// The generic `C` is the type of the incoming connection before it is upgraded. -/// -/// > **Note**: The `upgrade` method of this trait uses `self` and not `&self` or `&mut self`. -/// > This has been designed so that you would implement this trait on `&Foo` or -/// > `&mut Foo` instead of directly on `Foo`. -pub trait ConnectionUpgrade { - /// Iterator returned by `protocol_names`. - type NamesIter: Iterator; - /// Type that serves as an identifier for the protocol. This type only exists to be returned - /// by the `NamesIter` and then be passed to `upgrade`. - /// - /// This is only useful on implementations that dispatch between multiple possible upgrades. - /// Any basic implementation will probably just use the `()` type. - type UpgradeIdentifier; - - /// Returns the name of the protocols to advertise to the remote. - fn protocol_names(&self) -> Self::NamesIter; - - /// Type of the stream that has been upgraded. Generally wraps around `C` and `Self`. - /// - /// > **Note**: For upgrades that add an intermediary layer (such as `secio` or `multiplex`), - /// > this associated type must implement `AsyncRead + AsyncWrite`. - type Output; - /// Type of the future that will resolve to `Self::Output`. - type Future: Future; - - /// This method is called after protocol negotiation has been performed. - /// - /// Because performing the upgrade may not be instantaneous (e.g. it may require a handshake), - /// this function returns a future instead of the direct output. - fn upgrade(self, socket: C, id: Self::UpgradeIdentifier, ty: Endpoint) -> Self::Future; -} diff --git a/misc/core-derive/src/lib.rs b/misc/core-derive/src/lib.rs index 80241ba4..0d5f3421 100644 --- a/misc/core-derive/src/lib.rs +++ b/misc/core-derive/src/lib.rs @@ -68,29 +68,44 @@ fn build_struct(ast: &DeriveInput, data_struct: &DataStruct) -> TokenStream { quote!{#n} }; + let output_types = { + let mut start = 1; + // Avoid collisions. + while ast.generics.type_params().any(|tp| tp.ident.to_string() == format!("TOut{}", start)) { + start += 1; + } + data_struct.fields.iter() + .filter(|x| !is_ignored(x)) + .enumerate() + .map(move |(i, _)| Ident::new(&format!("TOut{}", start + i), name.span())) + .collect::>() + }; + // Build the generics. let impl_generics = { let tp = ast.generics.type_params(); let lf = ast.generics.lifetimes(); let cst = ast.generics.const_params(); - quote!{<#(#lf,)* #(#tp,)* #(#cst,)* #substream_generic>} + let out = output_types.clone(); + quote!{<#(#lf,)* #(#tp,)* #(#cst,)* #substream_generic, #(#out),*>} }; // Build the `where ...` clause of the trait implementation. let where_clause = { - let mut additional = data_struct.fields.iter().flat_map(|field| { - if is_ignored(&field) { - return vec![]; - } - - let ty = &field.ty; - vec![ - quote!{#ty: #trait_to_impl}, - quote!{<#ty as #trait_to_impl>::ProtocolsHandler: #protocols_handler}, - // Note: this bound is required because of https://github.com/rust-lang/rust/issues/55697 - quote!{<<#ty as #trait_to_impl>::ProtocolsHandler as #protocols_handler>::Protocol: ::libp2p::core::ConnectionUpgrade<#substream_generic>}, - ] - }).collect::>(); + let mut additional = data_struct.fields.iter() + .filter(|x| !is_ignored(x)) + .zip(output_types) + .flat_map(|(field, out)| { + let ty = &field.ty; + vec![ + quote!{#ty: #trait_to_impl}, + quote!{<#ty as #trait_to_impl>::ProtocolsHandler: #protocols_handler}, + // Note: this bound is required because of https://github.com/rust-lang/rust/issues/55697 + quote!{<<#ty as #trait_to_impl>::ProtocolsHandler as #protocols_handler>::InboundProtocol: ::libp2p::core::InboundUpgrade<#substream_generic, Output = #out>}, + quote!{<<#ty as #trait_to_impl>::ProtocolsHandler as #protocols_handler>::OutboundProtocol: ::libp2p::core::OutboundUpgrade<#substream_generic, Output = #out>}, + ] + }) + .collect::>(); additional.push(quote!{#substream_generic: ::libp2p::tokio_io::AsyncRead}); additional.push(quote!{#substream_generic: ::libp2p::tokio_io::AsyncWrite}); diff --git a/muxers/mplex/src/lib.rs b/muxers/mplex/src/lib.rs index 12ff37bc..c1626ba0 100644 --- a/muxers/mplex/src/lib.rs +++ b/muxers/mplex/src/lib.rs @@ -36,7 +36,12 @@ use std::{cmp, iter, mem}; use std::io::{Error as IoError, ErrorKind as IoErrorKind}; use std::sync::{atomic::AtomicUsize, atomic::Ordering, Arc}; use bytes::Bytes; -use core::{ConnectionUpgrade, Endpoint, StreamMuxer, muxing::Shutdown}; +use core::{ + Endpoint, + StreamMuxer, + muxing::Shutdown, + upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo} +}; use parking_lot::Mutex; use fnv::{FnvHashMap, FnvHashSet}; use futures::prelude::*; @@ -91,6 +96,31 @@ impl MplexConfig { self.max_buffer_behaviour = behaviour; self } + + #[inline] + fn upgrade(self, i: C, endpoint: Endpoint) -> Multiplex + where + C: AsyncRead + AsyncWrite + { + let max_buffer_len = self.max_buffer_len; + Multiplex { + inner: Mutex::new(MultiplexInner { + error: Ok(()), + inner: executor::spawn(Framed::new(i, codec::Codec::new()).fuse()), + config: self, + buffer: Vec::with_capacity(cmp::min(max_buffer_len, 512)), + opened_substreams: Default::default(), + next_outbound_stream_id: if endpoint == Endpoint::Dialer { 0 } else { 1 }, + notifier_read: Arc::new(Notifier { + to_notify: Mutex::new(Default::default()), + }), + notifier_write: Arc::new(Notifier { + to_notify: Mutex::new(Default::default()), + }), + is_shutdown: false + }) + } + } } impl Default for MplexConfig { @@ -117,39 +147,9 @@ pub enum MaxBufferBehaviour { Block, } -impl ConnectionUpgrade for MplexConfig -where - C: AsyncRead + AsyncWrite, -{ - type Output = Multiplex; - type Future = future::FutureResult; - type UpgradeIdentifier = (); - type NamesIter = iter::Once<(Bytes, ())>; - - #[inline] - fn upgrade(self, i: C, _: (), endpoint: Endpoint) -> Self::Future { - let max_buffer_len = self.max_buffer_len; - - let out = Multiplex { - inner: Mutex::new(MultiplexInner { - error: Ok(()), - inner: executor::spawn(Framed::new(i, codec::Codec::new()).fuse()), - config: self, - buffer: Vec::with_capacity(cmp::min(max_buffer_len, 512)), - opened_substreams: Default::default(), - next_outbound_stream_id: if endpoint == Endpoint::Dialer { 0 } else { 1 }, - notifier_read: Arc::new(Notifier { - to_notify: Mutex::new(Default::default()), - }), - notifier_write: Arc::new(Notifier { - to_notify: Mutex::new(Default::default()), - }), - is_shutdown: false - }) - }; - - future::ok(out) - } +impl UpgradeInfo for MplexConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; #[inline] fn protocol_names(&self) -> Self::NamesIter { @@ -157,6 +157,32 @@ where } } +impl InboundUpgrade for MplexConfig +where + C: AsyncRead + AsyncWrite, +{ + type Output = Multiplex; + type Error = IoError; + type Future = future::FutureResult; + + fn upgrade_inbound(self, socket: C, _: Self::UpgradeId) -> Self::Future { + future::ok(self.upgrade(socket, Endpoint::Listener)) + } +} + +impl OutboundUpgrade for MplexConfig +where + C: AsyncRead + AsyncWrite, +{ + type Output = Multiplex; + type Error = IoError; + type Future = future::FutureResult; + + fn upgrade_outbound(self, socket: C, _: Self::UpgradeId) -> Self::Future { + future::ok(self.upgrade(socket, Endpoint::Dialer)) + } +} + /// Multiplexer. Implements the `StreamMuxer` trait. pub struct Multiplex { inner: Mutex>, diff --git a/muxers/yamux/src/lib.rs b/muxers/yamux/src/lib.rs index 0ec13430..29816b96 100644 --- a/muxers/yamux/src/lib.rs +++ b/muxers/yamux/src/lib.rs @@ -22,14 +22,14 @@ extern crate bytes; extern crate futures; #[macro_use] extern crate log; -extern crate libp2p_core as core; +extern crate libp2p_core; extern crate parking_lot; extern crate tokio_io; extern crate yamux; use bytes::Bytes; -use core::{Endpoint, muxing::Shutdown}; use futures::{future::{self, FutureResult}, prelude::*}; +use libp2p_core::{muxing::Shutdown, upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}}; use parking_lot::Mutex; use std::{io, iter}; use std::io::{Error as IoError}; @@ -47,7 +47,7 @@ where } } -impl core::StreamMuxer for Yamux +impl libp2p_core::StreamMuxer for Yamux where C: AsyncRead + AsyncWrite + 'static { @@ -134,27 +134,38 @@ impl Default for Config { } } -impl core::ConnectionUpgrade for Config -where - C: AsyncRead + AsyncWrite + 'static, -{ - type UpgradeIdentifier = (); - type NamesIter = iter::Once<(Bytes, ())>; +impl UpgradeInfo for Config { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; fn protocol_names(&self) -> Self::NamesIter { iter::once((Bytes::from("/yamux/1.0.0"), ())) } +} +impl InboundUpgrade for Config +where + C: AsyncRead + AsyncWrite + 'static, +{ type Output = Yamux; + type Error = io::Error; type Future = FutureResult, io::Error>; - fn upgrade(self, i: C, _: (), end: Endpoint) -> Self::Future { - let mode = match end { - Endpoint::Dialer => yamux::Mode::Client, - Endpoint::Listener => yamux::Mode::Server - }; - - future::ok(Yamux::new(i, self.0, mode)) + fn upgrade_inbound(self, i: C, _: Self::UpgradeId) -> Self::Future { + future::ok(Yamux::new(i, self.0, yamux::Mode::Server)) + } +} + +impl OutboundUpgrade for Config +where + C: AsyncRead + AsyncWrite + 'static, +{ + type Output = Yamux; + type Error = io::Error; + type Future = FutureResult, io::Error>; + + fn upgrade_outbound(self, i: C, _: Self::UpgradeId) -> Self::Future { + future::ok(Yamux::new(i, self.0, yamux::Mode::Client)) } } diff --git a/protocols/floodsub/src/handler.rs b/protocols/floodsub/src/handler.rs index 563743f3..a5b1160e 100644 --- a/protocols/floodsub/src/handler.rs +++ b/protocols/floodsub/src/handler.rs @@ -18,10 +18,12 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::protocol::{FloodsubCodec, FloodsubConfig, FloodsubRpc}; use futures::prelude::*; -use libp2p_core::nodes::{NodeHandlerEndpoint, ProtocolsHandler, ProtocolsHandlerEvent}; -use libp2p_core::ConnectionUpgrade; -use protocol::{FloodsubCodec, FloodsubConfig, FloodsubRpc}; +use libp2p_core::{ + nodes::{ProtocolsHandler, ProtocolsHandlerEvent}, + upgrade::{InboundUpgrade, OutboundUpgrade} +}; use smallvec::SmallVec; use std::{fmt, io}; use tokio_codec::Framed; @@ -103,32 +105,39 @@ where type InEvent = FloodsubRpc; type OutEvent = FloodsubRpc; type Substream = TSubstream; - type Protocol = FloodsubConfig; + type InboundProtocol = FloodsubConfig; + type OutboundProtocol = FloodsubConfig; type OutboundOpenInfo = FloodsubRpc; #[inline] - fn listen_protocol(&self) -> Self::Protocol { + fn listen_protocol(&self) -> Self::InboundProtocol { self.config.clone() } - fn inject_fully_negotiated( + #[inline] + fn dialer_protocol(&self) -> Self::OutboundProtocol { + self.config.clone() + } + + fn inject_fully_negotiated_inbound( &mut self, - protocol: >::Output, - endpoint: NodeHandlerEndpoint, + protocol: >::Output ) { if self.shutting_down { - return; + return () } + self.substreams.push(SubstreamState::WaitingInput(protocol)) + } - match endpoint { - NodeHandlerEndpoint::Dialer(message) => { - self.substreams - .push(SubstreamState::PendingSend(protocol, message)); - } - NodeHandlerEndpoint::Listener => { - self.substreams.push(SubstreamState::WaitingInput(protocol)); - } + fn inject_fully_negotiated_outbound( + &mut self, + protocol: >::Output, + message: Self::OutboundOpenInfo + ) { + if self.shutting_down { + return () } + self.substreams.push(SubstreamState::PendingSend(protocol, message)) } #[inline] @@ -154,7 +163,7 @@ where fn poll( &mut self, ) -> Poll< - Option>, + Option>, io::Error, > { if !self.send_queue.is_empty() { diff --git a/protocols/floodsub/src/protocol.rs b/protocols/floodsub/src/protocol.rs index 31856786..e418284b 100644 --- a/protocols/floodsub/src/protocol.rs +++ b/protocols/floodsub/src/protocol.rs @@ -19,10 +19,10 @@ // DEALINGS IN THE SOFTWARE. use bytes::{BufMut, Bytes, BytesMut}; +use crate::rpc_proto; use futures::future; -use libp2p_core::{ConnectionUpgrade, Endpoint, PeerId}; +use libp2p_core::{InboundUpgrade, OutboundUpgrade, UpgradeInfo, PeerId}; use protobuf::Message as ProtobufMessage; -use rpc_proto; use std::{io, iter}; use tokio_codec::{Decoder, Encoder, Framed}; use tokio_io::{AsyncRead, AsyncWrite}; @@ -41,29 +41,41 @@ impl FloodsubConfig { } } -impl ConnectionUpgrade for FloodsubConfig -where - TSocket: AsyncRead + AsyncWrite, -{ - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); +impl UpgradeInfo for FloodsubConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::once(("/floodsub/1.0.0".into(), ())) } +} +impl InboundUpgrade for FloodsubConfig +where + TSocket: AsyncRead + AsyncWrite, +{ type Output = Framed; - type Future = future::FutureResult; + type Error = io::Error; + type Future = future::FutureResult; #[inline] - fn upgrade(self, socket: TSocket, _: Self::UpgradeIdentifier, _: Endpoint) -> Self::Future { - future::ok(Framed::new( - socket, - FloodsubCodec { - length_prefix: Default::default(), - }, - )) + fn upgrade_inbound(self, socket: TSocket, _: Self::UpgradeId) -> Self::Future { + future::ok(Framed::new(socket, FloodsubCodec { length_prefix: Default::default() })) + } +} + +impl OutboundUpgrade for FloodsubConfig +where + TSocket: AsyncRead + AsyncWrite, +{ + type Output = Framed; + type Error = io::Error; + type Future = future::FutureResult; + + #[inline] + fn upgrade_outbound(self, socket: TSocket, _: Self::UpgradeId) -> Self::Future { + future::ok(Framed::new(socket, FloodsubCodec { length_prefix: Default::default() })) } } diff --git a/protocols/identify/src/id_transport.rs b/protocols/identify/src/id_transport.rs index b83318ca..63644dda 100644 --- a/protocols/identify/src/id_transport.rs +++ b/protocols/identify/src/id_transport.rs @@ -21,8 +21,11 @@ //! Contains the `IdentifyTransport` type. use futures::prelude::*; -use libp2p_core::{Endpoint, Multiaddr, PeerId, PublicKey, Transport, muxing, upgrade::apply}; -use protocol::{IdentifyOutput, IdentifyProtocolConfig}; +use libp2p_core::{ + Multiaddr, PeerId, PublicKey, muxing, Transport, + upgrade::{self, OutboundUpgradeApply, UpgradeError} +}; +use protocol::{RemoteInfo, IdentifyProtocolConfig}; use std::io::{Error as IoError, ErrorKind as IoErrorKind}; use std::mem; use std::sync::Arc; @@ -71,26 +74,7 @@ where #[inline] fn listen_on(self, addr: Multiaddr) -> Result<(Self::Listener, Multiaddr), (Self, Multiaddr)> { - let (listener, new_addr) = match self.transport.listen_on(addr) { - Ok((l, a)) => (l, a), - Err((inner, addr)) => { - let id = IdentifyTransport { - transport: inner, - }; - return Err((id, addr)); - } - }; - - let listener = listener - .map(move |(upgrade, remote_addr)| { - let upgr = upgrade - .and_then(move |muxer| { - IdRetriever::new(muxer, IdentifyProtocolConfig, Endpoint::Listener) - }); - (Box::new(upgr) as Box + Send>, remote_addr) - }); - - Ok((Box::new(listener) as Box<_>, new_addr)) + Err((self, addr)) } #[inline] @@ -107,7 +91,7 @@ where }; let dial = dial.and_then(move |muxer| { - IdRetriever::new(muxer, IdentifyProtocolConfig, Endpoint::Dialer) + IdRetriever::new(muxer, IdentifyProtocolConfig).map_err(|e| e.into_io_error()) }); Ok(Box::new(dial) as Box<_>) @@ -126,9 +110,7 @@ where TMuxer: muxing::StreamMuxer + Send + Sync + 'static, TMuxer::Substream: Send, { /// Internal state. - state: IdRetrieverState, - /// Whether we're dialing or listening. - endpoint: Endpoint, + state: IdRetrieverState } enum IdRetrieverState @@ -138,7 +120,7 @@ where TMuxer: muxing::StreamMuxer + Send + Sync + 'static, /// We are in the process of opening a substream with the remote. OpeningSubstream(Arc, muxing::OutboundSubstreamRefWrapFuture>, IdentifyProtocolConfig), /// We opened the substream and are currently negotiating the identify protocol. - NegotiatingIdentify(Arc, apply::UpgradeApplyFuture>, IdentifyProtocolConfig>), + NegotiatingIdentify(Arc, OutboundUpgradeApply>, IdentifyProtocolConfig>), /// We retreived the remote's public key and are ready to yield it when polled again. Finishing(Arc, PublicKey), /// Something bad happend, or the `Future` is finished, and shouldn't be polled again. @@ -150,13 +132,12 @@ where TMuxer: muxing::StreamMuxer + Send + Sync + 'static, TMuxer::Substream: Send, { /// Creates a new `IdRetriever` ready to be polled. - fn new(muxer: TMuxer, config: IdentifyProtocolConfig, endpoint: Endpoint) -> Self { + fn new(muxer: TMuxer, config: IdentifyProtocolConfig) -> Self { let muxer = Arc::new(muxer); let opening = muxing::outbound_from_ref_and_wrap(muxer.clone()); IdRetriever { - state: IdRetrieverState::OpeningSubstream(muxer, opening, config), - endpoint, + state: IdRetrieverState::OpeningSubstream(muxer, opening, config) } } } @@ -166,7 +147,7 @@ where TMuxer: muxing::StreamMuxer + Send + Sync + 'static, TMuxer::Substream: Send, { type Item = (PeerId, TMuxer); - type Error = IoError; + type Error = UpgradeError; fn poll(&mut self) -> Poll { // This loop is here so that we can continue polling until we're ready. @@ -177,28 +158,24 @@ where TMuxer: muxing::StreamMuxer + Send + Sync + 'static, IdRetrieverState::OpeningSubstream(muxer, mut opening, config) => { match opening.poll() { Ok(Async::Ready(Some(substream))) => { - let upgrade = apply::apply(substream, config, self.endpoint); - self.state = IdRetrieverState::NegotiatingIdentify(muxer, upgrade); + let upgrade = upgrade::apply_outbound(substream, config); + self.state = IdRetrieverState::NegotiatingIdentify(muxer, upgrade) }, Ok(Async::Ready(None)) => { - return Err(IoError::new(IoErrorKind::Other, "remote refused our identify attempt")); - }, + return Err(UpgradeError::Apply(IoError::new(IoErrorKind::Other, "remote refused our identify attempt"))) + } Ok(Async::NotReady) => { self.state = IdRetrieverState::OpeningSubstream(muxer, opening, config); return Ok(Async::NotReady); }, - Err(err) => return Err(err), + Err(err) => return Err(UpgradeError::Apply(err)) } }, IdRetrieverState::NegotiatingIdentify(muxer, mut nego) => { match nego.poll() { - Ok(Async::Ready(IdentifyOutput::RemoteInfo { info, .. })) => { + Ok(Async::Ready(RemoteInfo { info, .. })) => { self.state = IdRetrieverState::Finishing(muxer, info.public_key); }, - Ok(Async::Ready(IdentifyOutput::Sender { .. })) => { - unreachable!("IdentifyOutput::Sender can never be the output from \ - the dialing side"); - }, Ok(Async::NotReady) => { self.state = IdRetrieverState::NegotiatingIdentify(muxer, nego); return Ok(Async::NotReady); diff --git a/protocols/identify/src/lib.rs b/protocols/identify/src/lib.rs index 0b368baa..b8253842 100644 --- a/protocols/identify/src/lib.rs +++ b/protocols/identify/src/lib.rs @@ -70,7 +70,6 @@ extern crate fnv; extern crate futures; extern crate libp2p_peerstore; extern crate libp2p_core; -#[macro_use] extern crate log; extern crate multiaddr; extern crate parking_lot; @@ -87,7 +86,7 @@ pub use self::listen_handler::IdentifyListenHandler; pub use self::listen_layer::IdentifyListen; pub use self::periodic_id_handler::{PeriodicIdentification, PeriodicIdentificationEvent}; pub use self::periodic_id_layer::{PeriodicIdentifyBehaviour, PeriodicIdentifyBehaviourEvent}; -pub use self::protocol::{IdentifyInfo, IdentifyOutput}; +pub use self::protocol::{IdentifyInfo, RemoteInfo}; pub use self::protocol::{IdentifyProtocolConfig, IdentifySender}; mod id_transport; diff --git a/protocols/identify/src/listen_handler.rs b/protocols/identify/src/listen_handler.rs index 9eddbd05..78951b34 100644 --- a/protocols/identify/src/listen_handler.rs +++ b/protocols/identify/src/listen_handler.rs @@ -18,15 +18,16 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::{IdentifySender, IdentifyProtocolConfig}; use futures::prelude::*; -use libp2p_core::nodes::handled_node::NodeHandlerEndpoint; -use libp2p_core::nodes::protocols_handler::{ProtocolsHandler, ProtocolsHandlerEvent}; -use libp2p_core::ConnectionUpgrade; +use libp2p_core::{ + nodes::protocols_handler::{ProtocolsHandler, ProtocolsHandlerEvent}, + upgrade::{DeniedUpgrade, InboundUpgrade} +}; use smallvec::SmallVec; use std::io; use tokio_io::{AsyncRead, AsyncWrite}; -use void::Void; -use {IdentifySender, IdentifyOutput, IdentifyProtocolConfig}; +use void::{Void, unreachable}; /// Protocol handler that identifies the remote at a regular period. pub struct IdentifyListenHandler { @@ -59,28 +60,29 @@ where type InEvent = Void; type OutEvent = IdentifySender; type Substream = TSubstream; - type Protocol = IdentifyProtocolConfig; + type InboundProtocol = IdentifyProtocolConfig; + type OutboundProtocol = DeniedUpgrade; type OutboundOpenInfo = (); #[inline] - fn listen_protocol(&self) -> Self::Protocol { + fn listen_protocol(&self) -> Self::InboundProtocol { self.config.clone() } - fn inject_fully_negotiated( + #[inline] + fn dialer_protocol(&self) -> Self::OutboundProtocol { + DeniedUpgrade + } + + fn inject_fully_negotiated_inbound( &mut self, - protocol: >::Output, - endpoint: NodeHandlerEndpoint, + protocol: >::Output ) { - match protocol { - IdentifyOutput::Sender { sender } => { - debug_assert!(if let NodeHandlerEndpoint::Listener = endpoint { true } else { false }); - self.pending_result.push(sender); - } - IdentifyOutput::RemoteInfo { .. } => unreachable!( - "RemoteInfo can only be produced if we dial the protocol, but we never do that" - ), - } + self.pending_result.push(protocol) + } + + fn inject_fully_negotiated_outbound(&mut self, protocol: Void, _: Self::OutboundOpenInfo) { + unreachable(protocol) } #[inline] @@ -102,7 +104,7 @@ where ) -> Poll< Option< ProtocolsHandlerEvent< - Self::Protocol, + Self::OutboundProtocol, Self::OutboundOpenInfo, Self::OutEvent, >, diff --git a/protocols/identify/src/periodic_id_handler.rs b/protocols/identify/src/periodic_id_handler.rs index bc54e9c0..ae2a2f0e 100644 --- a/protocols/identify/src/periodic_id_handler.rs +++ b/protocols/identify/src/periodic_id_handler.rs @@ -18,18 +18,16 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::{RemoteInfo, IdentifyProtocolConfig}; use futures::prelude::*; -use libp2p_core::nodes::handled_node::NodeHandlerEndpoint; -use libp2p_core::nodes::protocols_handler::{ProtocolsHandler, ProtocolsHandlerEvent}; -use libp2p_core::upgrade::{self, toggleable::Toggleable}; -use libp2p_core::{ConnectionUpgrade, Multiaddr}; -use std::io; -use std::marker::PhantomData; -use std::time::{Duration, Instant}; +use libp2p_core::{ + nodes::protocols_handler::{ProtocolsHandler, ProtocolsHandlerEvent}, + upgrade::{self, DeniedUpgrade, OutboundUpgrade, Toggleable} +}; +use std::{io, marker::PhantomData, time::{Duration, Instant}}; use tokio_io::{AsyncRead, AsyncWrite}; use tokio_timer::Delay; -use void::Void; -use {IdentifyInfo, IdentifyOutput, IdentifyProtocolConfig}; +use void::{Void, unreachable}; /// Delay between the moment we connect and the first time we identify. const DELAY_TO_FIRST_ID: Duration = Duration::from_millis(500); @@ -59,13 +57,7 @@ pub struct PeriodicIdentification { #[derive(Debug)] pub enum PeriodicIdentificationEvent { /// We obtained identification information from the remote - Identified { - /// Information of the remote. - info: IdentifyInfo, - /// Address the remote observes us as. - observed_addr: Multiaddr, - }, - + Identified(RemoteInfo), /// Failed to identify the remote. IdentificationError(io::Error), } @@ -90,36 +82,30 @@ where type InEvent = Void; type OutEvent = PeriodicIdentificationEvent; type Substream = TSubstream; - type Protocol = Toggleable; + type InboundProtocol = DeniedUpgrade; + type OutboundProtocol = Toggleable; type OutboundOpenInfo = (); #[inline] - fn listen_protocol(&self) -> Self::Protocol { - let mut upgrade = self.config.clone(); - upgrade.disable(); - upgrade + fn listen_protocol(&self) -> Self::InboundProtocol { + DeniedUpgrade } - fn inject_fully_negotiated( + #[inline] + fn dialer_protocol(&self) -> Self::OutboundProtocol { + self.config.clone() + } + + fn inject_fully_negotiated_inbound(&mut self, protocol: Void) { + unreachable(protocol) + } + + fn inject_fully_negotiated_outbound( &mut self, - protocol: >::Output, - _endpoint: NodeHandlerEndpoint, + protocol: >::Output, + _info: Self::OutboundOpenInfo, ) { - match protocol { - IdentifyOutput::RemoteInfo { - info, - observed_addr, - } => { - self.pending_result = Some(PeriodicIdentificationEvent::Identified { - info, - observed_addr, - }); - } - IdentifyOutput::Sender { .. } => unreachable!( - "Sender can only be produced if we listen for the identify \ - protocol ; however we disable it in listen_protocol" - ), - } + self.pending_result = Some(PeriodicIdentificationEvent::Identified(protocol)) } #[inline] @@ -146,7 +132,7 @@ where ) -> Poll< Option< ProtocolsHandlerEvent< - Self::Protocol, + Self::OutboundProtocol, Self::OutboundOpenInfo, PeriodicIdentificationEvent, >, diff --git a/protocols/identify/src/periodic_id_layer.rs b/protocols/identify/src/periodic_id_layer.rs index 54a36c83..955bfe88 100644 --- a/protocols/identify/src/periodic_id_layer.rs +++ b/protocols/identify/src/periodic_id_layer.rs @@ -65,15 +65,12 @@ where event: ::OutEvent, ) { match event { - PeriodicIdentificationEvent::Identified { - info, - observed_addr, - } => { + PeriodicIdentificationEvent::Identified(remote) => { self.events .push_back(PeriodicIdentifyBehaviourEvent::Identified { peer_id: peer_id, - info: info, - observed_addr: observed_addr, + info: remote.info, + observed_addr: remote.observed_addr, }); } _ => (), // TODO: exhaustive pattern diff --git a/protocols/identify/src/protocol.rs b/protocols/identify/src/protocol.rs index f6ec3f20..7405f5e6 100644 --- a/protocols/identify/src/protocol.rs +++ b/protocols/identify/src/protocol.rs @@ -19,8 +19,12 @@ // DEALINGS IN THE SOFTWARE. use bytes::{Bytes, BytesMut}; -use futures::{future, Future, Sink, Stream}; -use libp2p_core::{ConnectionUpgrade, Endpoint, Multiaddr, PublicKey}; +use futures::{future::{self, FutureResult}, Future, Sink, Stream}; +use libp2p_core::{ + Multiaddr, PublicKey, + upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo} +}; +use log::{debug, trace}; use protobuf::Message as ProtobufMessage; use protobuf::parse_from_bytes as protobuf_parse_from_bytes; use protobuf::RepeatedField; @@ -35,22 +39,14 @@ use unsigned_varint::codec; #[derive(Debug, Clone)] pub struct IdentifyProtocolConfig; -/// Output of the connection upgrade. -pub enum IdentifyOutput { - /// We obtained information from the remote. Happens when we are the dialer. - RemoteInfo { - /// Information about the remote. - info: IdentifyInfo, - /// Address the remote sees for us. - observed_addr: Multiaddr, - }, +#[derive(Debug, Clone)] +pub struct RemoteInfo { + /// Information about the remote. + pub info: IdentifyInfo, + /// Address the remote sees for us. + pub observed_addr: Multiaddr, - /// We opened a connection to the remote and need to send it information. Happens when we are - /// the listener. - Sender { - /// Object used to send identify info to the client. - sender: IdentifySender, - }, + _priv: () } /// Object used to send back information to the client. @@ -64,11 +60,9 @@ where { /// Sends back information to the client. Returns a future that is signalled whenever the /// info have been sent. - pub fn send( - self, - info: IdentifyInfo, - observed_addr: &Multiaddr, - ) -> Box + Send + 'a> { + pub fn send(self, info: IdentifyInfo, observed_addr: &Multiaddr) + -> Box + Send + 'a> + { debug!("Sending identify info to client"); trace!("Sending: {:?}", info); @@ -108,67 +102,73 @@ pub struct IdentifyInfo { pub listen_addrs: Vec, /// Protocols supported by the node, e.g. `/ipfs/ping/1.0.0`. pub protocols: Vec, + + _priv: () } -impl ConnectionUpgrade for IdentifyProtocolConfig -where - C: AsyncRead + AsyncWrite + Send + 'static, -{ - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); - type Output = IdentifyOutput; - type Future = Box + Send>; +impl UpgradeInfo for IdentifyProtocolConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::once((Bytes::from("/ipfs/id/1.0.0"), ())) } +} - fn upgrade(self, socket: C, _: (), ty: Endpoint) -> Self::Future { - trace!("Upgrading connection as {:?}", ty); +impl InboundUpgrade for IdentifyProtocolConfig +where + C: AsyncRead + AsyncWrite + Send + 'static, +{ + type Output = IdentifySender; + type Error = IoError; + type Future = FutureResult; + fn upgrade_inbound(self, socket: C, _: ()) -> Self::Future { + trace!("Upgrading inbound connection"); let socket = Framed::new(socket, codec::UviBytes::default()); + let sender = IdentifySender { inner: socket }; + future::ok(sender) + } +} - match ty { - Endpoint::Dialer => { - let future = socket - .into_future() - .map(|(msg, _)| msg) - .map_err(|(err, _)| err) - .and_then(|msg| { - debug!("Received identify message"); +impl OutboundUpgrade for IdentifyProtocolConfig +where + C: AsyncRead + AsyncWrite + Send + 'static, +{ + type Output = RemoteInfo; + type Error = IoError; + type Future = Box + Send>; - if let Some(msg) = msg { - let (info, observed_addr) = match parse_proto_msg(msg) { - Ok(v) => v, - Err(err) => { - debug!("Failed to parse protobuf message; error = {:?}", err); - return Err(err.into()); - } - }; - - trace!("Remote observes us as {:?}", observed_addr); - trace!("Information received: {:?}", info); - - Ok(IdentifyOutput::RemoteInfo { - info, - observed_addr: observed_addr.clone(), - }) - } else { - debug!("Identify protocol stream closed before receiving info"); - Err(IoErrorKind::InvalidData.into()) + fn upgrade_outbound(self, socket: C, _: ()) -> Self::Future { + let socket = Framed::new(socket, codec::UviBytes::::default()); + let future = socket + .into_future() + .map(|(msg, _)| msg) + .map_err(|(err, _)| err) + .and_then(|msg| { + debug!("Received identify message"); + if let Some(msg) = msg { + let (info, observed_addr) = match parse_proto_msg(msg) { + Ok(v) => v, + Err(err) => { + debug!("Failed to parse protobuf message; error = {:?}", err); + return Err(err.into()); } - }); - - Box::new(future) as Box<_> - } - - Endpoint::Listener => { - let sender = IdentifySender { inner: socket }; - let future = future::ok(IdentifyOutput::Sender { sender }); - Box::new(future) as Box<_> - } - } + }; + trace!("Remote observes us as {:?}", observed_addr); + trace!("Information received: {:?}", info); + Ok(RemoteInfo { + info, + observed_addr: observed_addr.clone(), + _priv: () + }) + } else { + debug!("Identify protocol stream closed before receiving info"); + Err(IoErrorKind::InvalidData.into()) + } + }); + Box::new(future) as Box<_> } } @@ -199,6 +199,7 @@ fn parse_proto_msg(msg: BytesMut) -> Result<(IdentifyInfo, Multiaddr), IoError> agent_version: msg.take_agentVersion(), listen_addrs: listen_addrs, protocols: msg.take_protocols().into_vec(), + _priv: () }; Ok((info, observed_addr)) @@ -216,10 +217,10 @@ mod tests { use self::tokio::runtime::current_thread::Runtime; use self::libp2p_tcp_transport::TcpConfig; use futures::{Future, Stream}; - use libp2p_core::{PublicKey, Transport}; + use libp2p_core::{PublicKey, Transport, upgrade::{apply_outbound, apply_inbound}}; use std::sync::mpsc; use std::thread; - use {IdentifyInfo, IdentifyOutput, IdentifyProtocolConfig}; + use {IdentifyInfo, RemoteInfo, IdentifyProtocolConfig}; #[test] fn correct_transfer() { @@ -229,19 +230,23 @@ mod tests { let (tx, rx) = mpsc::channel(); let bg_thread = thread::spawn(move || { - let transport = TcpConfig::new().with_upgrade(IdentifyProtocolConfig); + let transport = TcpConfig::new(); let (listener, addr) = transport .listen_on("/ip4/127.0.0.1/tcp/0".parse().unwrap()) .unwrap(); + tx.send(addr).unwrap(); let future = listener .into_future() .map_err(|(err, _)| err) .and_then(|(client, _)| client.unwrap().0) - .and_then(|identify| match identify { - IdentifyOutput::Sender { sender, .. } => sender.send( + .and_then(|socket| { + apply_inbound(socket, IdentifyProtocolConfig).map_err(|e| e.into_io_error()) + }) + .and_then(|sender| { + sender.send( IdentifyInfo { public_key: PublicKey::Ed25519(vec![1, 2, 3, 4, 5, 7]), protocol_version: "proto_version".to_owned(), @@ -251,47 +256,34 @@ mod tests { "/ip6/::1/udp/1000".parse().unwrap(), ], protocols: vec!["proto1".to_string(), "proto2".to_string()], + _priv: () }, &"/ip4/100.101.102.103/tcp/5000".parse().unwrap(), - ), - _ => panic!(), + ) }); let mut rt = Runtime::new().unwrap(); let _ = rt.block_on(future).unwrap(); }); - let transport = TcpConfig::new().with_upgrade(IdentifyProtocolConfig); + let transport = TcpConfig::new(); - let future = transport - .dial(rx.recv().unwrap()) + let future = transport.dial(rx.recv().unwrap()) .unwrap_or_else(|_| panic!()) - .and_then(|identify| match identify { - IdentifyOutput::RemoteInfo { - info, - observed_addr, - } => { - assert_eq!( - observed_addr, - "/ip4/100.101.102.103/tcp/5000".parse().unwrap() - ); - assert_eq!(info.public_key, PublicKey::Ed25519(vec![1, 2, 3, 4, 5, 7])); - assert_eq!(info.protocol_version, "proto_version"); - assert_eq!(info.agent_version, "agent_version"); - assert_eq!( - info.listen_addrs, - &[ - "/ip4/80.81.82.83/tcp/500".parse().unwrap(), - "/ip6/::1/udp/1000".parse().unwrap() - ] - ); - assert_eq!( - info.protocols, - &["proto1".to_string(), "proto2".to_string()] - ); - Ok(()) - } - _ => panic!(), + .and_then(|socket| { + apply_outbound(socket, IdentifyProtocolConfig).map_err(|e| e.into_io_error()) + }) + .and_then(|RemoteInfo { info, observed_addr, .. }| { + assert_eq!(observed_addr, "/ip4/100.101.102.103/tcp/5000".parse().unwrap()); + assert_eq!(info.public_key, PublicKey::Ed25519(vec![1, 2, 3, 4, 5, 7])); + assert_eq!(info.protocol_version, "proto_version"); + assert_eq!(info.agent_version, "agent_version"); + assert_eq!(info.listen_addrs, + &["/ip4/80.81.82.83/tcp/500".parse().unwrap(), + "/ip6/::1/udp/1000".parse().unwrap()]); + assert_eq!(info.protocols, &["proto1".to_string(), "proto2".to_string()]); + Ok(()) }); + let mut rt = Runtime::new().unwrap(); let _ = rt.block_on(future).unwrap(); bg_thread.join().unwrap(); diff --git a/protocols/kad/src/high_level.rs b/protocols/kad/src/high_level.rs index e086d7a2..9e5f04f8 100644 --- a/protocols/kad/src/high_level.rs +++ b/protocols/kad/src/high_level.rs @@ -23,6 +23,7 @@ use futures::{future, Future, IntoFuture, stream, Stream}; use kad_server::KadConnecController; use kbucket::{KBucketsTable, KBucketsPeerId}; use libp2p_core::PeerId; +use log::{debug, trace}; use protocol; use rand; use smallvec::SmallVec; diff --git a/protocols/kad/src/kad_server.rs b/protocols/kad/src/kad_server.rs index b4180502..95eb380a 100644 --- a/protocols/kad/src/kad_server.rs +++ b/protocols/kad/src/kad_server.rs @@ -36,7 +36,8 @@ use bytes::Bytes; use futures::sync::{mpsc, oneshot}; use futures::{future, Future, Sink, stream, Stream}; -use libp2p_core::{ConnectionUpgrade, Endpoint, PeerId}; +use libp2p_core::{PeerId, upgrade::{InboundUpgrade, UpgradeInfo}}; +use log::{debug, warn}; use multihash::Multihash; use protocol::{self, KadMsg, KademliaProtocolConfig, KadPeer}; use std::collections::VecDeque; @@ -64,7 +65,17 @@ impl KadConnecConfig { } } -impl ConnectionUpgrade for KadConnecConfig +impl UpgradeInfo for KadConnecConfig { + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; + type UpgradeId = (); + + #[inline] + fn protocol_names(&self) -> Self::NamesIter { + self.raw_proto.protocol_names() + } +} + +impl InboundUpgrade for KadConnecConfig where C: AsyncRead + AsyncWrite + Send + 'static, // TODO: 'static :-/ { @@ -72,22 +83,14 @@ where KadConnecController, Box + Send>, ); - type Future = future::Map<>::Future, fn(>::Output) -> Self::Output>; - type NamesIter = iter::Once<(Bytes, ())>; - type UpgradeIdentifier = (); + type Error = IoError; + type Future = future::Map<>::Future, fn(>::Output) -> Self::Output>; #[inline] - fn protocol_names(&self) -> Self::NamesIter { - ConnectionUpgrade::::protocol_names(&self.raw_proto) - } - - #[inline] - fn upgrade(self, incoming: C, id: (), endpoint: Endpoint) -> Self::Future { + fn upgrade_inbound(self, incoming: C, id: Self::UpgradeId) -> Self::Future { self.raw_proto - .upgrade(incoming, id, endpoint) - .map:: _, _>(move |connec| { - build_from_sink_stream(connec) - }) + .upgrade_inbound(incoming, id) + .map(build_from_sink_stream) } } diff --git a/protocols/kad/src/lib.rs b/protocols/kad/src/lib.rs index 883bd1b1..49b1281a 100644 --- a/protocols/kad/src/lib.rs +++ b/protocols/kad/src/lib.rs @@ -66,7 +66,6 @@ extern crate futures; extern crate libp2p_identify; extern crate libp2p_ping; extern crate libp2p_core; -#[macro_use] extern crate log; extern crate multiaddr; extern crate multihash; diff --git a/protocols/kad/src/protocol.rs b/protocols/kad/src/protocol.rs index bad4fe75..808fa27c 100644 --- a/protocols/kad/src/protocol.rs +++ b/protocols/kad/src/protocol.rs @@ -27,7 +27,10 @@ use bytes::{Bytes, BytesMut}; use futures::{future, sink, Sink, stream, Stream}; -use libp2p_core::{ConnectionUpgrade, Endpoint, Multiaddr, PeerId}; +use libp2p_core::{ + Multiaddr, PeerId, + upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo} +}; use multihash::Multihash; use protobuf::{self, Message}; use protobuf_structs; @@ -128,22 +131,40 @@ impl Into for KadPeer { #[derive(Debug, Default, Copy, Clone)] pub struct KademliaProtocolConfig; -impl ConnectionUpgrade for KademliaProtocolConfig -where - C: AsyncRead + AsyncWrite + 'static, // TODO: 'static :-/ -{ - type Output = KadStreamSink; - type Future = future::FutureResult; - type NamesIter = iter::Once<(Bytes, ())>; - type UpgradeIdentifier = (); +impl UpgradeInfo for KademliaProtocolConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::once(("/ipfs/kad/1.0.0".into(), ())) } +} + +impl InboundUpgrade for KademliaProtocolConfig +where + C: AsyncRead + AsyncWrite + 'static, // TODO: 'static :-/ +{ + type Output = KadStreamSink; + type Error = IoError; + type Future = future::FutureResult; #[inline] - fn upgrade(self, incoming: C, _: (), _: Endpoint) -> Self::Future { + fn upgrade_inbound(self, incoming: C, _: Self::UpgradeId) -> Self::Future { + future::ok(kademlia_protocol(incoming)) + } +} + +impl OutboundUpgrade for KademliaProtocolConfig +where + C: AsyncRead + AsyncWrite + 'static, // TODO: 'static :-/ +{ + type Output = KadStreamSink; + type Error = IoError; + type Future = future::FutureResult; + + #[inline] + fn upgrade_outbound(self, incoming: C, _: Self::UpgradeId) -> Self::Future { future::ok(kademlia_protocol(incoming)) } } @@ -151,9 +172,7 @@ where type KadStreamSink = stream::AndThen>>, IoError>, KadMsg, fn(KadMsg) -> Result, IoError>, Result, IoError>>, fn(BytesMut) -> Result, Result>; // Upgrades a socket to use the Kademlia protocol. -fn kademlia_protocol( - socket: S, -) -> KadStreamSink +fn kademlia_protocol(socket: S) -> KadStreamSink where S: AsyncRead + AsyncWrite, { @@ -485,6 +504,7 @@ mod tests { let (listener, addr) = transport .listen_on("/ip4/127.0.0.1/tcp/0".parse().unwrap()) .unwrap(); + tx.send(addr).unwrap(); let future = listener diff --git a/protocols/observed/src/lib.rs b/protocols/observed/src/lib.rs index be17b1a4..919e3ebf 100644 --- a/protocols/observed/src/lib.rs +++ b/protocols/observed/src/lib.rs @@ -30,19 +30,67 @@ extern crate unsigned_varint; use bytes::Bytes; use futures::{future, prelude::*}; -use libp2p_core::{ConnectionUpgrade, Endpoint, Multiaddr}; +use libp2p_core::{Multiaddr, upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}}; use std::{io, iter}; use tokio_codec::{FramedRead, FramedWrite}; use tokio_io::{AsyncRead, AsyncWrite}; use unsigned_varint::codec::UviBytes; -/// The output, this connection upgrade produces. -pub enum Output { - /// As `Dialer`, we get our own externally observed address. - Address(Multiaddr), - /// As `Listener`, we return a sender which allows reporting the observed - /// address the client. - Sender(Sender) +/// The connection upgrade type to retrieve or report externally visible addresses. +pub struct Observed {} + +impl Observed { + pub fn new() -> Self { + Observed {} + } +} + +impl UpgradeInfo for Observed { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; + + fn protocol_names(&self) -> Self::NamesIter { + iter::once((Bytes::from("/paritytech/observed-address/0.1.0"), ())) + } +} + +impl InboundUpgrade for Observed +where + C: AsyncRead + AsyncWrite + Send + 'static +{ + type Output = Sender; + type Error = io::Error; + type Future = Box + Send>; + + fn upgrade_inbound(self, conn: C, _: ()) -> Self::Future { + let io = FramedWrite::new(conn, UviBytes::default()); + Box::new(future::ok(Sender { io })) + } +} + +impl OutboundUpgrade for Observed +where + C: AsyncRead + AsyncWrite + Send + 'static +{ + type Output = Multiaddr; + type Error = io::Error; + type Future = Box + Send>; + + fn upgrade_outbound(self, conn: C, _: ()) -> Self::Future { + let io = FramedRead::new(conn, UviBytes::default()); + let future = io.into_future() + .map_err(|(e, _): (io::Error, FramedRead)| e) + .and_then(move |(bytes, _)| { + if let Some(b) = bytes { + let ma = Multiaddr::from_bytes(b.to_vec()) + .map_err(|e| io::Error::new(io::ErrorKind::InvalidData, e))?; + Ok(ma) + } else { + Err(io::ErrorKind::InvalidData.into()) + } + }); + Box::new(future) + } } /// `Sender` allows reporting back the observed address to the remote endpoint. @@ -57,58 +105,11 @@ impl Sender { } } -/// The connection upgrade type to retrieve or report externally visible addresses. -pub struct Observed {} - -impl Observed { - pub fn new() -> Self { - Observed {} - } -} - -impl ConnectionUpgrade for Observed -where - C: AsyncRead + AsyncWrite + Send + 'static -{ - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); - type Output = Output; - type Future = Box + Send>; - - fn protocol_names(&self) -> Self::NamesIter { - iter::once((Bytes::from("/paritytech/observed-address/0.1.0"), ())) - } - - fn upgrade(self, conn: C, _: (), role: Endpoint) -> Self::Future { - match role { - Endpoint::Dialer => { - let io = FramedRead::new(conn, UviBytes::default()); - let future = io.into_future() - .map_err(|(e, _): (io::Error, FramedRead)| e) - .and_then(move |(bytes, _)| { - if let Some(b) = bytes { - let ma = Multiaddr::from_bytes(b.to_vec()) - .map_err(|e| io::Error::new(io::ErrorKind::InvalidData, e))?; - Ok(Output::Address(ma)) - } else { - Err(io::ErrorKind::InvalidData.into()) - } - }); - Box::new(future) - } - Endpoint::Listener => { - let io = FramedWrite::new(conn, UviBytes::default()); - Box::new(future::ok(Output::Sender(Sender { io }))) - } - } - } -} - #[cfg(test)] mod tests { extern crate tokio; - use libp2p_core::{ConnectionUpgrade, Endpoint, Multiaddr}; + use libp2p_core::{Multiaddr, upgrade::{InboundUpgrade, OutboundUpgrade}}; use self::tokio::runtime::current_thread; use self::tokio::net::{TcpListener, TcpStream}; use super::*; @@ -125,28 +126,18 @@ mod tests { .into_future() .map_err(|(e, _)| e.into()) .and_then(move |(conn, _)| { - Observed::new().upgrade(conn.unwrap(), (), Endpoint::Listener) + Observed::new().upgrade_inbound(conn.unwrap(), ()) }) - .and_then(move |output| { - match output { - Output::Sender(s) => s.send_address(observed_addr1), - Output::Address(_) => unreachable!() - } - }); + .and_then(move |sender| sender.send_address(observed_addr1)); let client = TcpStream::connect(&server_addr) .map_err(|e| e.into()) .and_then(|conn| { - Observed::new().upgrade(conn, (), Endpoint::Dialer) + Observed::new().upgrade_outbound(conn, ()) }) - .map(move |output| { - match output { - Output::Address(addr) => { - eprintln!("{} {}", addr, observed_addr2); - assert_eq!(addr, observed_addr2) - } - _ => unreachable!() - } + .map(move |addr| { + eprintln!("{} {}", addr, observed_addr2); + assert_eq!(addr, observed_addr2) }); current_thread::block_on_all(future::lazy(move || { diff --git a/protocols/ping/src/dial_handler.rs b/protocols/ping/src/dial_handler.rs index bb7ae0b0..30c1c8f2 100644 --- a/protocols/ping/src/dial_handler.rs +++ b/protocols/ping/src/dial_handler.rs @@ -20,25 +20,26 @@ use futures::prelude::*; use libp2p_core::{ - nodes::{NodeHandlerEndpoint, ProtocolsHandler, ProtocolsHandlerEvent}, - upgrade::toggleable, - ConnectionUpgrade, + OutboundUpgrade, + nodes::{ProtocolsHandler, ProtocolsHandlerEvent}, + upgrade::{self, DeniedUpgrade} }; -use protocol::{Ping, PingDialer, PingOutput}; +use log::warn; +use protocol::{Ping, PingDialer}; use std::{ io, mem, time::{Duration, Instant}, }; use tokio_io::{AsyncRead, AsyncWrite}; use tokio_timer::Delay; -use void::Void; +use void::{Void, unreachable}; /// Protocol handler that handles pinging the remote at a regular period. /// /// If the remote doesn't respond, produces `Unresponsive` and closes the connection. pub struct PeriodicPingHandler { /// Configuration for the ping protocol. - ping_config: toggleable::Toggleable>, + ping_config: upgrade::Toggleable>, /// State of the outgoing ping. out_state: OutState, @@ -126,7 +127,7 @@ impl PeriodicPingHandler { let ping_timeout = Duration::from_secs(30); PeriodicPingHandler { - ping_config: toggleable::toggleable(Default::default()), + ping_config: upgrade::toggleable(Default::default()), out_state: OutState::NeedToOpen { expires: Delay::new(Instant::now() + ping_timeout), }, @@ -151,36 +152,36 @@ where type InEvent = Void; type OutEvent = OutEvent; type Substream = TSubstream; - type Protocol = toggleable::Toggleable>; + type InboundProtocol = DeniedUpgrade; + type OutboundProtocol = upgrade::Toggleable>; type OutboundOpenInfo = (); #[inline] - fn listen_protocol(&self) -> Self::Protocol { - let mut config = self.ping_config; - config.disable(); - config + fn listen_protocol(&self) -> Self::InboundProtocol { + DeniedUpgrade } - fn inject_fully_negotiated( + #[inline] + fn dialer_protocol(&self) -> Self::OutboundProtocol { + self.ping_config + } + + fn inject_fully_negotiated_inbound(&mut self, protocol: Void) { + unreachable(protocol) + } + + fn inject_fully_negotiated_outbound( &mut self, - protocol: >::Output, - _endpoint: NodeHandlerEndpoint, + mut substream: >::Output, + _info: Self::OutboundOpenInfo ) { - match protocol { - PingOutput::Pinger(mut substream) => { - debug_assert!(_endpoint.is_dialer()); - match mem::replace(&mut self.out_state, OutState::Poisoned) { - OutState::Upgrading { expires } => { - // We always upgrade with the intent of immediately pinging. - substream.ping(Instant::now()); - self.out_state = OutState::WaitingForPong { substream, expires }; - } - _ => (), - } - } - PingOutput::Ponger(_) => { - debug_assert!(false, "Received an unexpected incoming ping substream"); + match mem::replace(&mut self.out_state, OutState::Poisoned) { + OutState::Upgrading { expires } => { + // We always upgrade with the intent of immediately pinging. + substream.ping(Instant::now()); + self.out_state = OutState::WaitingForPong { substream, expires }; } + _ => (), } } @@ -218,7 +219,7 @@ where fn poll( &mut self, ) -> Poll< - Option>, + Option>, io::Error, > { // Shortcut for polling a `tokio_timer::Delay` diff --git a/protocols/ping/src/lib.rs b/protocols/ping/src/lib.rs index 1f07713f..cf5e1cb3 100644 --- a/protocols/ping/src/lib.rs +++ b/protocols/ping/src/lib.rs @@ -56,38 +56,35 @@ //! extern crate tokio; //! //! use futures::{Future, Stream}; -//! use libp2p_ping::protocol::{Ping, PingOutput}; -//! use libp2p_core::Transport; +//! use libp2p_core::{transport::Transport, upgrade::apply_outbound}; +//! use libp2p_ping::protocol::Ping; //! use tokio::runtime::current_thread::Runtime; //! //! # fn main() { -//! let ping_finished_future = libp2p_tcp_transport::TcpConfig::new() -//! .with_upgrade(Ping::default()) -//! .dial("127.0.0.1:12345".parse::().unwrap()).unwrap_or_else(|_| panic!()) -//! .and_then(|out| { -//! match out { -//! PingOutput::Ponger(processing) => Box::new(processing) as Box + Send>, -//! PingOutput::Pinger(mut pinger) => { -//! pinger.ping(()); -//! let f = pinger.into_future().map(|_| ()).map_err(|(err, _)| err); -//! Box::new(f) as Box + Send> -//! }, -//! } +//! let ping_dialer = libp2p_tcp_transport::TcpConfig::new() +//! .and_then(|socket, _| { +//! apply_outbound(socket, Ping::default()).map_err(|e| e.into_io_error()) +//! }) +//! .dial("/ip4/127.0.0.1/tcp/12345".parse::().unwrap()).unwrap_or_else(|_| panic!()) +//! .and_then(|mut pinger| { +//! pinger.ping(()); +//! let f = pinger.into_future() +//! .map(|_| println!("received pong")) +//! .map_err(|(e, _)| e); +//! Box::new(f) as Box + Send> //! }); //! //! // Runs until the ping arrives. //! let mut rt = Runtime::new().unwrap(); -//! let _ = rt.block_on(ping_finished_future).unwrap(); +//! let _ = rt.block_on(ping_dialer).unwrap(); //! # } //! ``` //! extern crate arrayvec; extern crate bytes; -#[macro_use] extern crate futures; extern crate libp2p_core; -#[macro_use] extern crate log; extern crate multistream_select; extern crate parking_lot; diff --git a/protocols/ping/src/listen_handler.rs b/protocols/ping/src/listen_handler.rs index 04c0c6a6..78bb9824 100644 --- a/protocols/ping/src/listen_handler.rs +++ b/protocols/ping/src/listen_handler.rs @@ -21,13 +21,15 @@ use arrayvec::ArrayVec; use futures::prelude::*; use libp2p_core::{ - nodes::{NodeHandlerEndpoint, ProtocolsHandler, ProtocolsHandlerEvent}, - ConnectionUpgrade, + InboundUpgrade, + nodes::{ProtocolsHandler, ProtocolsHandlerEvent}, + upgrade::DeniedUpgrade }; -use protocol::{Ping, PingListener, PingOutput}; +use log::warn; +use protocol::{Ping, PingListener}; use std::io; use tokio_io::{AsyncRead, AsyncWrite}; -use void::Void; +use void::{Void, unreachable}; /// Handler for handling pings received from a remote. pub struct PingListenHandler { @@ -68,33 +70,32 @@ where type InEvent = Void; type OutEvent = Void; type Substream = TSubstream; - type Protocol = Ping<()>; + type InboundProtocol = Ping<()>; + type OutboundProtocol = DeniedUpgrade; type OutboundOpenInfo = (); #[inline] - fn listen_protocol(&self) -> Self::Protocol { + fn listen_protocol(&self) -> Self::InboundProtocol { self.ping_config } - fn inject_fully_negotiated( + #[inline] + fn dialer_protocol(&self) -> Self::OutboundProtocol { + DeniedUpgrade + } + + fn inject_fully_negotiated_inbound( &mut self, - protocol: >::Output, - _endpoint: NodeHandlerEndpoint, + protocol: >::Output ) { if self.shutdown { return; } + let _ = self.ping_in_substreams.try_push(protocol); + } - match protocol { - PingOutput::Pinger(_) => { - debug_assert!(false, "Received an unexpected outgoing ping substream"); - } - PingOutput::Ponger(listener) => { - debug_assert!(_endpoint.is_listener()); - // Try insert the element, but don't care if the list is full. - let _ = self.ping_in_substreams.try_push(listener); - } - } + fn inject_fully_negotiated_outbound(&mut self, protocol: Void, _info: Self::OutboundOpenInfo) { + unreachable(protocol) } #[inline] @@ -118,7 +119,7 @@ where fn poll( &mut self, ) -> Poll< - Option>, + Option>, io::Error, > { // Removes each substream one by one, and pushes them back if they're not ready (which diff --git a/protocols/ping/src/protocol.rs b/protocols/ping/src/protocol.rs index a748871c..a5ca043e 100644 --- a/protocols/ping/src/protocol.rs +++ b/protocols/ping/src/protocol.rs @@ -19,8 +19,9 @@ // DEALINGS IN THE SOFTWARE. use bytes::{BufMut, Bytes, BytesMut}; -use futures::{prelude::*, future::FutureResult, future::IntoFuture}; -use libp2p_core::{ConnectionUpgrade, Endpoint}; +use futures::{prelude::*, future::{self, FutureResult}, try_ready}; +use libp2p_core::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; +use log::debug; use rand::{distributions::Standard, prelude::*, rngs::EntropyRng}; use std::collections::VecDeque; use std::io::Error as IoError; @@ -42,71 +43,53 @@ impl Default for Ping { } } -/// Output of a `Ping` upgrade. -pub enum PingOutput { - /// We are on the dialing side. - Pinger(PingDialer), - /// We are on the listening side. - Ponger(PingListener), -} +impl UpgradeInfo for Ping { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; -impl ConnectionUpgrade for Ping -where - TSocket: AsyncRead + AsyncWrite, -{ - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); - - #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::once(("/ipfs/ping/1.0.0".into(), ())) } +} - type Output = PingOutput; - type Future = FutureResult; +impl InboundUpgrade for Ping +where + TSocket: AsyncRead + AsyncWrite, +{ + type Output = PingListener; + type Error = IoError; + type Future = FutureResult; #[inline] - fn upgrade( - self, - socket: TSocket, - _: Self::UpgradeIdentifier, - endpoint: Endpoint, - ) -> Self::Future { - let out = match endpoint { - Endpoint::Dialer => upgrade_as_dialer(socket), - Endpoint::Listener => upgrade_as_listener(socket), + fn upgrade_inbound(self, socket: TSocket, _: Self::UpgradeId) -> Self::Future { + let listener = PingListener { + inner: Framed::new(socket, Codec), + state: PingListenerState::Listening, }; - - Ok(out).into_future() + future::ok(listener) } } -/// Upgrades a connection from the dialer side. -fn upgrade_as_dialer(socket: TSocket) -> PingOutput -where TSocket: AsyncRead + AsyncWrite, +impl OutboundUpgrade for Ping +where + TSocket: AsyncRead + AsyncWrite, { - let dialer = PingDialer { - inner: Framed::new(socket, Codec), - need_writer_flush: false, - needs_close: false, - sent_pings: VecDeque::with_capacity(4), - rng: EntropyRng::default(), - pings_to_send: VecDeque::with_capacity(4), - }; + type Output = PingDialer; + type Error = IoError; + type Future = FutureResult; - PingOutput::Pinger(dialer) -} - -/// Upgrades a connection from the listener side. -fn upgrade_as_listener(socket: TSocket) -> PingOutput -where TSocket: AsyncRead + AsyncWrite, -{ - let listener = PingListener { - inner: Framed::new(socket, Codec), - state: PingListenerState::Listening, - }; - - PingOutput::Ponger(listener) + #[inline] + fn upgrade_outbound(self, socket: TSocket, _: Self::UpgradeId) -> Self::Future { + let dialer = PingDialer { + inner: Framed::new(socket, Codec), + need_writer_flush: false, + needs_close: false, + sent_pings: VecDeque::with_capacity(4), + rng: EntropyRng::default(), + pings_to_send: VecDeque::with_capacity(4), + }; + future::ok(dialer) + } } /// Sends pings and receives the pongs. @@ -342,9 +325,9 @@ mod tests { use self::tokio_tcp::TcpListener; use self::tokio_tcp::TcpStream; - use super::{Ping, PingOutput}; + use super::Ping; use futures::{Future, Stream}; - use libp2p_core::{ConnectionUpgrade, Endpoint}; + use libp2p_core::upgrade::{InboundUpgrade, OutboundUpgrade}; // TODO: rewrite tests with the MemoryTransport @@ -358,32 +341,18 @@ mod tests { .into_future() .map_err(|(e, _)| e.into()) .and_then(|(c, _)| { - Ping::<()>::default().upgrade( - c.unwrap(), - (), - Endpoint::Listener, - ) + Ping::<()>::default().upgrade_inbound(c.unwrap(), ()) }) - .and_then(|out| match out { - PingOutput::Ponger(service) => service, - _ => unreachable!(), - }); + .flatten(); let client = TcpStream::connect(&listener_addr) .map_err(|e| e.into()) .and_then(|c| { - Ping::<()>::default().upgrade( - c, - (), - Endpoint::Dialer, - ) + Ping::<()>::default().upgrade_outbound(c, ()) }) - .and_then(|out| match out { - PingOutput::Pinger(mut pinger) => { - pinger.ping(()); - pinger.into_future().map(|_| ()).map_err(|_| panic!()) - }, - _ => unreachable!(), + .and_then(|mut pinger| { + pinger.ping(()); + pinger.into_future().map(|_| ()).map_err(|_| panic!()) }) .map(|_| ()); @@ -402,39 +371,23 @@ mod tests { .into_future() .map_err(|(e, _)| e.into()) .and_then(|(c, _)| { - Ping::::default().upgrade( - c.unwrap(), - (), - Endpoint::Listener, - ) + Ping::::default().upgrade_inbound(c.unwrap(), ()) }) - .and_then(|out| match out { - PingOutput::Ponger(service) => service, - _ => unreachable!(), - }); + .flatten(); let client = TcpStream::connect(&listener_addr) .map_err(|e| e.into()) .and_then(|c| { - Ping::::default().upgrade( - c, - (), - Endpoint::Dialer, - ) + Ping::::default().upgrade_outbound(c, ()) }) - .and_then(|out| match out { - PingOutput::Pinger(mut pinger) => { - for n in 0..20 { - pinger.ping(n); - } - - pinger - .take(20) - .collect() - .map(|val| { assert_eq!(val, (0..20).collect::>()); }) - .map_err(|_| panic!()) - }, - _ => unreachable!(), + .and_then(|mut pinger| { + for n in 0..20 { + pinger.ping(n); + } + pinger.take(20) + .collect() + .map(|val| { assert_eq!(val, (0..20).collect::>()); }) + .map_err(|_| panic!()) }); let mut runtime = tokio::runtime::Runtime::new().unwrap(); diff --git a/protocols/plaintext/Cargo.toml b/protocols/plaintext/Cargo.toml new file mode 100644 index 00000000..f75d2f40 --- /dev/null +++ b/protocols/plaintext/Cargo.toml @@ -0,0 +1,12 @@ +[package] +name = "libp2p-plaintext" +version = "0.1.0" +authors = ["Parity Technologies "] +license = "MIT" + +[dependencies] +bytes = "0.4" +futures = "0.1" +libp2p-core = { path = "../../core" } +void = "1" + diff --git a/core/src/upgrade/plaintext.rs b/protocols/plaintext/src/lib.rs similarity index 63% rename from core/src/upgrade/plaintext.rs rename to protocols/plaintext/src/lib.rs index ac969853..d2a64370 100644 --- a/core/src/upgrade/plaintext.rs +++ b/protocols/plaintext/src/lib.rs @@ -1,4 +1,4 @@ -// Copyright 2017 Parity Technologies (UK) Ltd. +// Copyright 2018 Parity Technologies (UK) Ltd. // // Permission is hereby granted, free of charge, to any person obtaining a // copy of this software and associated documentation files (the "Software"), @@ -18,36 +18,46 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +extern crate bytes; +extern crate futures; +extern crate libp2p_core; +extern crate void; + use bytes::Bytes; use futures::future::{self, FutureResult}; -use std::{iter, io::Error as IoError}; -use tokio_io::{AsyncRead, AsyncWrite}; -use upgrade::{ConnectionUpgrade, Endpoint}; +use libp2p_core::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; +use std::iter; +use void::Void; -/// Implementation of the `ConnectionUpgrade` that negotiates the `/plaintext/1.0.0` protocol and -/// simply passes communications through without doing anything more. -/// -/// > **Note**: Generally used as an alternative to `secio` if a security layer is not desirable. -// TODO: move to a separate crate? #[derive(Debug, Copy, Clone)] pub struct PlainTextConfig; -impl ConnectionUpgrade for PlainTextConfig -where - C: AsyncRead + AsyncWrite, -{ - type Output = C; - type Future = FutureResult; - type UpgradeIdentifier = (); - type NamesIter = iter::Once<(Bytes, ())>; +impl UpgradeInfo for PlainTextConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; - #[inline] - fn upgrade(self, i: C, _: (), _: Endpoint) -> Self::Future { - future::ok(i) - } - - #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::once((Bytes::from("/plaintext/1.0.0"), ())) } } + +impl InboundUpgrade for PlainTextConfig { + type Output = C; + type Error = Void; + type Future = FutureResult; + + fn upgrade_inbound(self, i: C, _: Self::UpgradeId) -> Self::Future { + future::ok(i) + } +} + +impl OutboundUpgrade for PlainTextConfig { + type Output = C; + type Error = Void; + type Future = FutureResult; + + fn upgrade_outbound(self, i: C, _: Self::UpgradeId) -> Self::Future { + future::ok(i) + } +} + diff --git a/protocols/secio/src/lib.rs b/protocols/secio/src/lib.rs index e894835e..9d768c13 100644 --- a/protocols/secio/src/lib.rs +++ b/protocols/secio/src/lib.rs @@ -39,12 +39,13 @@ //! # fn main() { //! use futures::Future; //! use libp2p_secio::{SecioConfig, SecioKeyPair, SecioOutput}; -//! use libp2p_core::{Multiaddr, Transport, upgrade}; +//! use libp2p_core::{Multiaddr, upgrade::apply_inbound}; +//! use libp2p_core::transport::Transport; //! use libp2p_tcp_transport::TcpConfig; //! use tokio_io::io::write_all; //! use tokio::runtime::current_thread::Runtime; //! -//! let transport = TcpConfig::new() +//! let dialer = TcpConfig::new() //! .with_upgrade({ //! # let private_key = b""; //! //let private_key = include_bytes!("test-rsa-private-key.pk8"); @@ -52,17 +53,17 @@ //! //let public_key = include_bytes!("test-rsa-public-key.der").to_vec(); //! // See the documentation of `SecioKeyPair`. //! let keypair = SecioKeyPair::rsa_from_pkcs8(private_key, public_key).unwrap(); -//! let upgrade = SecioConfig::new(keypair); +//! SecioConfig::new(keypair) +//! }) +//! .map(|out: SecioOutput<_>, _| out.stream); //! -//! upgrade::map(upgrade, |out: SecioOutput<_>| out.stream) -//! }); -//! -//! let future = transport.dial("/ip4/127.0.0.1/tcp/12345".parse::().unwrap()) -//! .unwrap_or_else(|_| panic!("Unable to dial node")) +//! let future = dialer.dial("/ip4/127.0.0.1/tcp/12345".parse::().unwrap()) +//! .unwrap_or_else(|_| panic!("Unable to dial node")) //! .and_then(|connection| { //! // Sends "hello world" on the connection, will be encrypted. //! write_all(connection, "hello world") -//! }); +//! }) +//! .map_err(|e| panic!("error: {:?}", e)); //! //! let mut rt = Runtime::new().unwrap(); //! let _ = rt.block_on(future).unwrap(); @@ -119,7 +120,7 @@ use bytes::{Bytes, BytesMut}; use ed25519_dalek::Keypair as Ed25519KeyPair; use futures::stream::MapErr as StreamMapErr; use futures::{Future, Poll, Sink, StartSend, Stream}; -use libp2p_core::{PeerId, PublicKey}; +use libp2p_core::{PeerId, PublicKey, upgrade::{UpgradeInfo, InboundUpgrade, OutboundUpgrade}}; #[cfg(all(feature = "rsa", not(target_os = "emscripten")))] use ring::signature::RSAKeyPair; use rw_stream_sink::RwStreamSink; @@ -191,6 +192,22 @@ impl SecioConfig { self.digests_prop = Some(algo_support::digests_proposition(xs)); self } + + fn handshake(self, socket: T, _: ()) -> impl Future, Error=SecioError> + where + T: AsyncRead + AsyncWrite + Send + 'static + { + debug!("Starting secio upgrade"); + SecioMiddleware::handshake(socket, self) + .map(|(stream_sink, pubkey, ephemeral)| { + let mapped = stream_sink.map_err(map_err as fn(_) -> _); + SecioOutput { + stream: RwStreamSink::new(mapped), + remote_key: pubkey, + ephemeral_public_key: ephemeral + } + }) + } } /// Private and public keys of the local node. @@ -353,39 +370,38 @@ where pub ephemeral_public_key: Vec, } -impl libp2p_core::ConnectionUpgrade for SecioConfig -where - S: AsyncRead + AsyncWrite + Send + 'static, // TODO: 'static :( -{ - type Output = SecioOutput; - type Future = Box + Send>; - type NamesIter = iter::Once<(Bytes, ())>; - type UpgradeIdentifier = (); +impl UpgradeInfo for SecioConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; - #[inline] fn protocol_names(&self) -> Self::NamesIter { iter::once(("/secio/1.0.0".into(), ())) } +} - #[inline] - fn upgrade( - self, - incoming: S, - _: (), - _: libp2p_core::Endpoint, - ) -> Self::Future { - debug!("Starting secio upgrade"); +impl InboundUpgrade for SecioConfig +where + T: AsyncRead + AsyncWrite + Send + 'static +{ + type Output = SecioOutput; + type Error = SecioError; + type Future = Box + Send>; - let fut = SecioMiddleware::handshake(incoming, self); - let wrapped = fut.map(|(stream_sink, pubkey, ephemeral)| { - let mapped = stream_sink.map_err(map_err as fn(_) -> _); - SecioOutput { - stream: RwStreamSink::new(mapped), - remote_key: pubkey, - ephemeral_public_key: ephemeral, - } - }).map_err(map_err); - Box::new(wrapped) + fn upgrade_inbound(self, socket: T, id: Self::UpgradeId) -> Self::Future { + Box::new(self.handshake(socket, id)) + } +} + +impl OutboundUpgrade for SecioConfig +where + T: AsyncRead + AsyncWrite + Send + 'static +{ + type Output = SecioOutput; + type Error = SecioError; + type Future = Box + Send>; + + fn upgrade_outbound(self, socket: T, id: Self::UpgradeId) -> Self::Future { + Box::new(self.handshake(socket, id)) } } diff --git a/src/lib.rs b/src/lib.rs index 8c40d60b..9169c25e 100644 --- a/src/lib.rs +++ b/src/lib.rs @@ -164,8 +164,11 @@ mod transport_ext; pub mod simple; +pub use self::core::{ + Transport, PeerId, + upgrade::{InboundUpgrade, InboundUpgradeExt, OutboundUpgrade, OutboundUpgradeExt} +}; pub use libp2p_core_derive::NetworkBehaviour; -pub use self::core::{Transport, ConnectionUpgrade, PeerId}; pub use self::multiaddr::Multiaddr; pub use self::simple::SimpleProtocol; pub use self::transport_ext::TransportExt; diff --git a/src/simple.rs b/src/simple.rs index 42dbd789..5a99f38e 100644 --- a/src/simple.rs +++ b/src/simple.rs @@ -19,7 +19,7 @@ // DEALINGS IN THE SOFTWARE. use bytes::Bytes; -use core::upgrade::{ConnectionUpgrade, Endpoint}; +use core::upgrade::{InboundUpgrade, OutboundUpgrade, UpgradeInfo}; use futures::prelude::*; use std::{iter, io::Error as IoError, sync::Arc}; use tokio_io::{AsyncRead, AsyncWrite}; @@ -57,26 +57,48 @@ impl Clone for SimpleProtocol { } } -impl ConnectionUpgrade for SimpleProtocol +impl UpgradeInfo for SimpleProtocol { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; + + #[inline] + fn protocol_names(&self) -> Self::NamesIter { + iter::once((self.name.clone(), ())) + } +} + +impl InboundUpgrade for SimpleProtocol where C: AsyncRead + AsyncWrite, F: Fn(C) -> O, O: IntoFuture, O::Future: Send + 'static, { - type NamesIter = iter::Once<(Bytes, ())>; - type UpgradeIdentifier = (); - - #[inline] - fn protocol_names(&self) -> Self::NamesIter { - iter::once((self.name.clone(), ())) - } - type Output = O::Item; - type Future = Box + Send>; + type Error = IoError; + type Future = Box + Send>; #[inline] - fn upgrade(self, socket: C, _: (), _: Endpoint) -> Self::Future { + fn upgrade_inbound(self, socket: C, _: Self::UpgradeId) -> Self::Future { + let upgrade = &self.upgrade; + let fut = upgrade(socket).into_future().from_err(); + Box::new(fut) as Box<_> + } +} + +impl OutboundUpgrade for SimpleProtocol +where + C: AsyncRead + AsyncWrite, + F: Fn(C) -> O, + O: IntoFuture, + O::Future: Send + 'static, +{ + type Output = O::Item; + type Error = IoError; + type Future = Box + Send>; + + #[inline] + fn upgrade_outbound(self, socket: C, _: Self::UpgradeId) -> Self::Future { let upgrade = &self.upgrade; let fut = upgrade(socket).into_future().from_err(); Box::new(fut) as Box<_> diff --git a/transports/relay/Cargo.toml b/transports/relay/Cargo.toml index 0fa426b2..89bd5f4e 100644 --- a/transports/relay/Cargo.toml +++ b/transports/relay/Cargo.toml @@ -16,3 +16,4 @@ rand = "0.4" tokio-codec = "0.1" tokio-io = "0.1" unsigned-varint = { version = "0.2.1", features = ["codec"] } +void = "1" diff --git a/transports/relay/src/copy.rs b/transports/relay/src/copy.rs index 0038043d..389b861b 100644 --- a/transports/relay/src/copy.rs +++ b/transports/relay/src/copy.rs @@ -26,8 +26,8 @@ // Based on copy.rs from tokio-io in https://github.com/tokio-rs/tokio +use futures::{Future, Poll, try_ready}; use std::io; -use futures::{Future, Poll}; use tokio_io::{AsyncRead, AsyncWrite}; /// A future which will copy all data from a reader into a writer. diff --git a/transports/relay/src/error.rs b/transports/relay/src/error.rs new file mode 100644 index 00000000..0dca557f --- /dev/null +++ b/transports/relay/src/error.rs @@ -0,0 +1,76 @@ +// Copyright 2018 Parity Technologies (UK) Ltd. +// +// Permission is hereby granted, free of charge, to any person obtaining a +// copy of this software and associated documentation files (the "Software"), +// to deal in the Software without restriction, including without limitation +// the rights to use, copy, modify, merge, publish, distribute, sublicense, +// and/or sell copies of the Software, and to permit persons to whom the +// Software is furnished to do so, subject to the following conditions: +// +// The above copyright notice and this permission notice shall be included in +// all copies or substantial portions of the Software. +// +// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS +// OR IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY, +// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE +// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER +// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING +// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER +// DEALINGS IN THE SOFTWARE. + +use libp2p_core::{PeerId, upgrade::UpgradeError}; +use std::{fmt, io}; + +#[derive(Debug)] +pub enum RelayError { + Io(io::Error), + Upgrade(UpgradeError), + NoRelayFor(PeerId), + Message(&'static str), + #[doc(hidden)] + __Nonexhaustive +} + + +impl fmt::Display for RelayError +where + E: fmt::Display +{ + fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result { + match self { + RelayError::Io(e) => write!(f, "i/o error: {}", e), + RelayError::Upgrade(e) => write!(f, "upgrade error: {}", e), + RelayError::NoRelayFor(p) => write!(f, "no relay for peer: {:?}", p), + RelayError::Message(m) => write!(f, "{}", m), + RelayError::__Nonexhaustive => f.write_str("__Nonexhaustive") + } + } +} + +impl std::error::Error for RelayError +where + E: std::error::Error +{ + fn cause(&self) -> Option<&dyn std::error::Error> { + match self { + RelayError::Io(e) => Some(e), + RelayError::Upgrade(e) => Some(e), + RelayError::NoRelayFor(_) => None, + RelayError::Message(_) => None, + RelayError::__Nonexhaustive => None + } + } +} + +impl From for RelayError { + fn from(e: io::Error) -> Self { + RelayError::Io(e) + } +} + +impl From> for RelayError { + fn from(e: UpgradeError) -> Self { + RelayError::Upgrade(e) + } +} + diff --git a/transports/relay/src/lib.rs b/transports/relay/src/lib.rs index 495ad305..cbdc40ef 100644 --- a/transports/relay/src/lib.rs +++ b/transports/relay/src/lib.rs @@ -19,11 +19,9 @@ // DEALINGS IN THE SOFTWARE. extern crate bytes; -#[macro_use] extern crate futures; extern crate libp2p_peerstore as peerstore; -extern crate libp2p_core as core; -#[macro_use] +extern crate libp2p_core; extern crate log; extern crate multiaddr; extern crate protobuf; @@ -31,8 +29,10 @@ extern crate rand; extern crate tokio_codec; extern crate tokio_io; extern crate unsigned_varint; +extern crate void; mod copy; +mod error; mod message; mod protocol; mod transport; diff --git a/transports/relay/src/protocol.rs b/transports/relay/src/protocol.rs index 984ce4d7..976262b5 100644 --- a/transports/relay/src/protocol.rs +++ b/transports/relay/src/protocol.rs @@ -19,19 +19,27 @@ // DEALINGS IN THE SOFTWARE. use bytes::Bytes; -use copy; -use core::{ConnectionUpgrade, Endpoint, Transport}; +use crate::{ + copy, + error::RelayError, + message::{CircuitRelay, CircuitRelay_Peer, CircuitRelay_Status, CircuitRelay_Type}, + utility::{io_err, is_success, status, Io, Peer} +}; use futures::{stream, future::{self, Either::{A, B}, FutureResult}, prelude::*}; -use message::{CircuitRelay, CircuitRelay_Peer, CircuitRelay_Status, CircuitRelay_Type}; +use libp2p_core::{ + transport::Transport, + upgrade::{apply_outbound, InboundUpgrade, OutboundUpgrade, UpgradeInfo} +}; +use log::debug; use peerstore::{PeerAccess, PeerId, Peerstore}; use std::{io, iter, ops::Deref}; use tokio_io::{AsyncRead, AsyncWrite}; -use utility::{io_err, is_success, status, Io, Peer}; +use void::Void; #[derive(Debug, Clone)] pub struct RelayConfig { my_id: PeerId, - transport: T, + dialer: T, peers: P, // If `allow_relays` is false this node can only be used as a // destination but will not allow relaying streams to other @@ -49,7 +57,16 @@ pub enum Output { Sealed(Box + Send>) } -impl ConnectionUpgrade for RelayConfig +impl UpgradeInfo for RelayConfig { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; + + fn protocol_names(&self) -> Self::NamesIter { + iter::once((Bytes::from("/libp2p/relay/circuit/0.1.0"), ())) + } +} + +impl InboundUpgrade for RelayConfig where C: AsyncRead + AsyncWrite + Send + 'static, T: Transport + Clone + Send + 'static, @@ -61,34 +78,28 @@ where S: 'static, for<'a> &'a S: Peerstore { - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); - - fn protocol_names(&self) -> Self::NamesIter { - iter::once((Bytes::from("/libp2p/relay/circuit/0.1.0"), ())) - } - type Output = Output; - type Future = Box + Send>; + type Error = RelayError; + type Future = Box + Send>; - fn upgrade(self, conn: C, _: (), _: Endpoint) -> Self::Future { - let future = Io::new(conn).recv().and_then(move |(message, io)| { + fn upgrade_inbound(self, conn: C, _: ()) -> Self::Future { + let future = Io::new(conn).recv().from_err().and_then(move |(message, io)| { let msg = if let Some(m) = message { m } else { - return A(A(future::err(io_err("no message received")))) + return A(A(future::err(RelayError::Message("no message received")))) }; match msg.get_field_type() { CircuitRelay_Type::HOP if self.allow_relays => { // act as relay B(A(self.on_hop(msg, io).map(|fut| Output::Sealed(Box::new(fut))))) } CircuitRelay_Type::STOP => { // act as destination - B(B(self.on_stop(msg, io).map(Output::Stream))) + B(B(self.on_stop(msg, io).from_err().map(Output::Stream))) } other => { debug!("invalid message type: {:?}", other); let resp = status(CircuitRelay_Status::MALFORMED_MESSAGE); - A(B(io.send(resp).and_then(|_| Err(io_err("invalid message type"))))) + A(B(io.send(resp).from_err().and_then(|_| Err(RelayError::Message("invalid message type"))))) } } }); @@ -106,8 +117,8 @@ where P: Deref + Clone + 'static, for<'a> &'a S: Peerstore, { - pub fn new(my_id: PeerId, transport: T, peers: P) -> RelayConfig { - RelayConfig { my_id, transport, peers, allow_relays: true } + pub fn new(my_id: PeerId, dialer: T, peers: P) -> RelayConfig { + RelayConfig { my_id, dialer, peers, allow_relays: true } } pub fn allow_relays(&mut self, val: bool) { @@ -115,7 +126,7 @@ where } // HOP message handling (relay mode). - fn on_hop(self, mut msg: CircuitRelay, io: Io) -> impl Future, Error=io::Error> + fn on_hop(self, mut msg: CircuitRelay, io: Io) -> impl Future, Error=RelayError> where C: AsyncRead + AsyncWrite + 'static, { @@ -123,14 +134,14 @@ where peer } else { let msg = status(CircuitRelay_Status::HOP_SRC_MULTIADDR_INVALID); - return A(io.send(msg).and_then(|_| Err(io_err("invalid src address")))) + return A(io.send(msg).from_err().and_then(|_| Err(RelayError::Message("invalid src address")))) }; let mut dest = if let Some(peer) = Peer::from_message(msg.take_dstPeer()) { peer } else { let msg = status(CircuitRelay_Status::HOP_DST_MULTIADDR_INVALID); - return B(A(io.send(msg).and_then(|_| Err(io_err("invalid dest address"))))) + return B(A(io.send(msg).from_err().and_then(|_| Err(RelayError::Message("invalid dest address"))))) }; if dest.addrs.is_empty() { @@ -142,13 +153,12 @@ where let stop = stop_message(&from, &dest); - let transport = self.transport.with_upgrade(TrivialUpgrade); - let dest_id = dest.id; + let dialer = self.dialer; let future = stream::iter_ok(dest.addrs.into_iter()) .and_then(move |dest_addr| { - transport.clone().dial(dest_addr).map_err(|_| io_err("failed to dial")) + dialer.clone().dial(dest_addr).map_err(|_| RelayError::Message("could no dial addr")) }) - .and_then(|dial| dial) + .and_then(|outbound| outbound.from_err().and_then(|c| apply_outbound(c, TrivialUpgrade).from_err())) .then(|result| Ok(result.ok())) .filter_map(|result| result) .into_future() @@ -156,22 +166,24 @@ where .and_then(move |(ok, _stream)| { if let Some(c) = ok { // send STOP message to destination and expect back a SUCCESS message - let future = Io::new(c).send(stop) + let future = Io::new(c) + .send(stop) .and_then(Io::recv) + .from_err() .and_then(|(response, io)| { let rsp = match response { Some(m) => m, - None => return Err(io_err("no message from destination")) + None => return Err(RelayError::Message("no message from destination")) }; if is_success(&rsp) { Ok(io.into()) } else { - Err(io_err("no success response from relay")) + Err(RelayError::Message("no success response from relay")) } }); A(future) } else { - B(future::err(io_err(format!("could not dial to {:?}", dest_id)))) + B(future::err(RelayError::Message("could not dial peer"))) } }) // signal success or failure to source @@ -179,11 +191,11 @@ where match result { Ok(c) => { let msg = status(CircuitRelay_Status::SUCCESS); - A(io.send(msg).map(|io| (io.into(), c))) + A(io.send(msg).map(|io| (io.into(), c)).from_err()) } Err(e) => { let msg = status(CircuitRelay_Status::HOP_CANT_DIAL_DST); - B(io.send(msg).and_then(|_| Err(e))) + B(io.send(msg).from_err().and_then(|_| Err(e))) } } }) @@ -247,21 +259,24 @@ fn stop_message(from: &Peer, dest: &Peer) -> CircuitRelay { #[derive(Debug, Clone)] struct TrivialUpgrade; -impl ConnectionUpgrade for TrivialUpgrade -where - C: AsyncRead + AsyncWrite + 'static -{ - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); +impl UpgradeInfo for TrivialUpgrade { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; fn protocol_names(&self) -> Self::NamesIter { iter::once((Bytes::from("/libp2p/relay/circuit/0.1.0"), ())) } +} +impl OutboundUpgrade for TrivialUpgrade +where + C: AsyncRead + AsyncWrite + 'static +{ type Output = C; - type Future = FutureResult; + type Error = Void; + type Future = FutureResult; - fn upgrade(self, conn: C, _: (), _: Endpoint) -> Self::Future { + fn upgrade_outbound(self, conn: C, _: ()) -> Self::Future { future::ok(conn) } } @@ -269,21 +284,24 @@ where #[derive(Debug, Clone)] pub(crate) struct Source(pub(crate) CircuitRelay); -impl ConnectionUpgrade for Source -where - C: AsyncRead + AsyncWrite + Send + 'static, -{ - type NamesIter = iter::Once<(Bytes, Self::UpgradeIdentifier)>; - type UpgradeIdentifier = (); +impl UpgradeInfo for Source { + type UpgradeId = (); + type NamesIter = iter::Once<(Bytes, Self::UpgradeId)>; fn protocol_names(&self) -> Self::NamesIter { iter::once((Bytes::from("/libp2p/relay/circuit/0.1.0"), ())) } +} +impl OutboundUpgrade for Source +where + C: AsyncRead + AsyncWrite + Send + 'static, +{ type Output = C; - type Future = Box + Send>; + type Error = io::Error; + type Future = Box + Send>; - fn upgrade(self, conn: C, _: (), _: Endpoint) -> Self::Future { + fn upgrade_outbound(self, conn: C, _: ()) -> Self::Future { let future = Io::new(conn) .send(self.0) .and_then(Io::recv) diff --git a/transports/relay/src/transport.rs b/transports/relay/src/transport.rs index 0e216b85..27f8a3e6 100644 --- a/transports/relay/src/transport.rs +++ b/transports/relay/src/transport.rs @@ -18,16 +18,20 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. -use core::Transport; +use crate::{ + error::RelayError, + message::{CircuitRelay, CircuitRelay_Peer, CircuitRelay_Type}, + protocol, + utility::{Peer, RelayAddr} +}; use futures::{stream, prelude::*}; -use message::{CircuitRelay, CircuitRelay_Peer, CircuitRelay_Type}; +use libp2p_core::{transport::Transport, upgrade::apply_outbound}; +use log::{debug, info, trace}; use multiaddr::Multiaddr; use peerstore::{PeerAccess, PeerId, Peerstore}; -use protocol; use rand::{self, Rng}; use std::{io, iter::FromIterator, ops::Deref, sync::Arc}; use tokio_io::{AsyncRead, AsyncWrite}; -use utility::{io_err, Peer, RelayAddr}; #[derive(Debug, Clone)] pub struct RelayTransport { @@ -70,10 +74,10 @@ where RelayAddr::Address { relay, dest } => { if let Some(ref r) = relay { let f = self.relay_via(r, &dest).map_err(|this| (this, addr))?; - Ok(Box::new(f)) + Ok(Box::new(f.map_err(|e| io::Error::new(io::ErrorKind::Other, e)))) } else { let f = self.relay_to(&dest).map_err(|this| (this, addr))?; - Ok(Box::new(f)) + Ok(Box::new(f.map_err(|e| io::Error::new(io::ErrorKind::Other, e)))) } } } @@ -111,7 +115,7 @@ where } // Relay to destination over any available relay node. - fn relay_to(self, destination: &Peer) -> Result, Self> { + fn relay_to(self, destination: &Peer) -> Result>, Self> { trace!("relay_to {:?}", destination.id); let mut dials = Vec::new(); for relay in &*self.relays { @@ -142,14 +146,14 @@ where if let Some(out) = ok { Ok(out) } else { - Err(io_err(format!("no relay for {:?}", dest_peer))) + Err(RelayError::NoRelayFor(dest_peer)) } }); Ok(future) } // Relay to destination via the given peer. - fn relay_via(self, relay: &Peer, destination: &Peer) -> Result, Self> { + fn relay_via(self, relay: &Peer, destination: &Peer) -> Result>, Self> { trace!("relay_via {:?} to {:?}", relay.id, destination.id); let mut addresses = Vec::new(); @@ -171,10 +175,15 @@ where let relay = relay.clone(); let message = self.hop_message(destination); - let transport = self.transport.with_upgrade(protocol::Source(message)); + let upgrade = protocol::Source(message); + let dialer = self.transport; let future = stream::iter_ok(addresses.into_iter()) - .filter_map(move |addr| transport.clone().dial(addr).ok()) - .and_then(|dial| dial) + .filter_map(move |addr| dialer.clone().dial(addr).ok()) + .and_then(move |dial| { + let upgrade = upgrade.clone(); + dial.map_err(|_| RelayError::Message("could not dial")) + .and_then(move |c| apply_outbound(c, upgrade).from_err()) + }) .then(|result| Ok(result.ok())) .filter_map(|result| result) .into_future() @@ -186,7 +195,7 @@ where } None => { info!("failed to dial to {:?}", relay.id); - Err(io_err(format!("failed to dial to relay {:?}", relay.id))) + Err(RelayError::Message("failed to dial to relay")) } }); Ok(future) diff --git a/transports/relay/src/utility.rs b/transports/relay/src/utility.rs index bb201335..f78c17f0 100644 --- a/transports/relay/src/utility.rs +++ b/transports/relay/src/utility.rs @@ -18,8 +18,9 @@ // FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER // DEALINGS IN THE SOFTWARE. +use crate::message::{CircuitRelay, CircuitRelay_Peer, CircuitRelay_Status, CircuitRelay_Type}; use futures::{future::{self, Either}, prelude::*}; -use message::{CircuitRelay, CircuitRelay_Peer, CircuitRelay_Status, CircuitRelay_Type}; +use log::trace; use multiaddr::{Protocol, Multiaddr}; use peerstore::PeerId; use protobuf::{self, Message};