mirror of
https://github.com/fluencelabs/rust-libp2p
synced 2025-06-26 08:11:39 +00:00
libp2p-ping improvements. (#1049)
* libp2p-ping improvements. * re #950: Removes use of the `OneShotHandler`, but still sending each ping over a new substream, as seems to be intentional since #828. * re #842: Adds an integration test that exercises the ping behaviour through a Swarm, requiring the RTT to be below a threshold. This requires disabling Nagle's algorithm as it can interact badly with delayed ACKs (and has been observed to do so in the context of the new ping example and integration test). * re #864: Control of the inbound and outbound (sub)stream protocol upgrade timeouts has been moved from the `NodeHandlerWrapperBuilder` to the `ProtocolsHandler`. That may also alleviate the need for a custom timeout on an `OutboundSubstreamRequest` as a `ProtocolsHandler` is now free to adjust these timeouts over time. Other changes: * A new ping example. * Documentation improvements. * More documentation improvements. * Add PingPolicy and ensure no event is dropped. * Remove inbound_timeout/outbound_timeout. As per review comment, the inbound timeout is now configured as part of the `listen_protocol` and the outbound timeout as part of the `OutboundSubstreamRequest`. * Simplify and generalise. Generalise `ListenProtocol` to `SubstreamProtocol`, reusing it in the context of `ProtocolsHandlerEvent::OutboundSubstreamRequest`. * Doc comments for SubstreamProtocol. * Adapt to changes in master. * Relax upper bound for ping integration test rtt. For "slow" CI build machines?
This commit is contained in:
@ -21,107 +21,262 @@
|
||||
use crate::protocol;
|
||||
use futures::prelude::*;
|
||||
use libp2p_core::ProtocolsHandlerEvent;
|
||||
use libp2p_core::protocols_handler::{KeepAlive, OneShotHandler, ProtocolsHandler, ProtocolsHandlerUpgrErr};
|
||||
use std::{io, time::Duration, time::Instant};
|
||||
use libp2p_core::protocols_handler::{
|
||||
KeepAlive,
|
||||
SubstreamProtocol,
|
||||
ProtocolsHandler,
|
||||
ProtocolsHandlerUpgrErr,
|
||||
};
|
||||
use std::{io, num::NonZeroU32, time::{Duration, Instant}};
|
||||
use std::collections::VecDeque;
|
||||
use tokio_io::{AsyncRead, AsyncWrite};
|
||||
use tokio_timer::Delay;
|
||||
use void::Void;
|
||||
|
||||
/// Protocol handler that handles pinging the remote at a regular period and answering ping
|
||||
/// queries.
|
||||
/// The configuration for outbound pings.
|
||||
#[derive(Clone, Debug)]
|
||||
pub struct PingConfig {
|
||||
/// The timeout of an outbound ping.
|
||||
timeout: Duration,
|
||||
/// The duration between the last successful outbound or inbound ping
|
||||
/// and the next outbound ping.
|
||||
interval: Duration,
|
||||
/// The maximum number of failed outbound pings before the associated
|
||||
/// connection is deemed unhealthy, indicating to the `Swarm` that it
|
||||
/// should be closed.
|
||||
max_timeouts: NonZeroU32,
|
||||
/// The policy for outbound pings.
|
||||
policy: PingPolicy
|
||||
}
|
||||
|
||||
impl PingConfig {
|
||||
/// Creates a new `PingConfig` with the following default settings:
|
||||
///
|
||||
/// * [`PingConfig::with_interval`] 15s
|
||||
/// * [`PingConfig::with_timeout`] 20s
|
||||
/// * [`PingConfig::with_max_timeouts`] 1
|
||||
/// * [`PingConfig::with_policy`] [`PingPolicy::Always`]
|
||||
///
|
||||
/// These settings have the following effect:
|
||||
///
|
||||
/// * A ping is sent every 15 seconds on a healthy connection.
|
||||
/// * Every ping sent must yield a response within 20 seconds in order to
|
||||
/// be successful.
|
||||
/// * The duration of a single ping timeout without sending or receiving
|
||||
/// a pong is sufficient for the connection to be subject to being closed,
|
||||
/// i.e. the connection timeout is reset to 20 seconds from the current
|
||||
/// [`Instant`] upon a sent or received pong.
|
||||
///
|
||||
/// In general, every successfully sent or received pong resets the connection
|
||||
/// timeout, which is defined by
|
||||
/// ```raw
|
||||
/// max_timeouts * timeout
|
||||
/// ```raw
|
||||
/// relative to the current [`Instant`].
|
||||
///
|
||||
/// A sensible configuration should thus obey the invariant:
|
||||
/// ```raw
|
||||
/// max_timeouts * timeout > interval
|
||||
/// ```
|
||||
pub fn new() -> Self {
|
||||
Self {
|
||||
timeout: Duration::from_secs(20),
|
||||
interval: Duration::from_secs(15),
|
||||
max_timeouts: NonZeroU32::new(1).expect("1 != 0"),
|
||||
policy: PingPolicy::Always
|
||||
}
|
||||
}
|
||||
|
||||
/// Sets the ping timeout.
|
||||
pub fn with_timeout(mut self, d: Duration) -> Self {
|
||||
self.timeout = d;
|
||||
self
|
||||
}
|
||||
|
||||
/// Sets the ping interval.
|
||||
pub fn with_interval(mut self, d: Duration) -> Self {
|
||||
self.interval = d;
|
||||
self
|
||||
}
|
||||
|
||||
/// Sets the number of successive ping timeouts upon which the remote
|
||||
/// peer is considered unreachable and the connection closed.
|
||||
///
|
||||
/// > **Note**: Successful inbound pings from the remote peer can keep
|
||||
/// > the connection alive, even if outbound pings fail. I.e.
|
||||
/// > the connection is closed after `ping_timeout * max_timeouts`
|
||||
/// > only if in addition to failing outbound pings, no ping from
|
||||
/// > the remote is received within that time window.
|
||||
pub fn with_max_timeouts(mut self, n: NonZeroU32) -> Self {
|
||||
self.max_timeouts = n;
|
||||
self
|
||||
}
|
||||
|
||||
/// Sets the [`PingPolicy`] to use for outbound pings.
|
||||
pub fn with_policy(mut self, p: PingPolicy) -> Self {
|
||||
self.policy = p;
|
||||
self
|
||||
}
|
||||
}
|
||||
|
||||
/// The `PingPolicy` determines under what conditions an outbound ping
|
||||
/// is sent w.r.t. inbound pings.
|
||||
#[derive(Clone, Debug, PartialEq, Eq)]
|
||||
pub enum PingPolicy {
|
||||
/// Always send a ping in the configured interval, regardless
|
||||
/// of received pings.
|
||||
///
|
||||
/// This policy is appropriate e.g. if continuous measurement of
|
||||
/// the RTT to the remote is desired.
|
||||
Always,
|
||||
/// Only sent a ping as necessary to keep the connection alive.
|
||||
///
|
||||
/// This policy resets the local ping timer whenever an inbound ping
|
||||
/// is received, effectively letting the peer with the lower ping
|
||||
/// frequency drive the ping protocol. Hence, to avoid superfluous ping
|
||||
/// traffic, the ping interval of the peers should ideally not be the
|
||||
/// same when using this policy, e.g. through randomization.
|
||||
///
|
||||
/// This policy is appropriate if the ping protocol is only used
|
||||
/// as an application-layer connection keep-alive, without a need
|
||||
/// for measuring the round-trip times on both peers, as it tries
|
||||
/// to keep the ping traffic to a minimum.
|
||||
KeepAlive
|
||||
}
|
||||
|
||||
/// The result of an inbound or outbound ping.
|
||||
pub type PingResult = Result<PingSuccess, PingFailure>;
|
||||
|
||||
/// The successful result of processing an inbound or outbound ping.
|
||||
#[derive(Debug)]
|
||||
pub enum PingSuccess {
|
||||
/// Received a ping and sent back a pong.
|
||||
Pong,
|
||||
/// Sent a ping and received back a pong.
|
||||
///
|
||||
/// Includes the round-trip time.
|
||||
Ping { rtt: Duration },
|
||||
}
|
||||
|
||||
/// An outbound ping failure.
|
||||
#[derive(Debug)]
|
||||
pub enum PingFailure {
|
||||
/// The ping timed out, i.e. no response was received within the
|
||||
/// configured ping timeout.
|
||||
Timeout,
|
||||
/// The ping failed for reasons other than a timeout.
|
||||
Other { error: Box<dyn std::error::Error + Send + 'static> }
|
||||
}
|
||||
|
||||
/// Protocol handler that handles pinging the remote at a regular period
|
||||
/// and answering ping queries.
|
||||
///
|
||||
/// If the remote doesn't respond, produces an error that closes the connection.
|
||||
pub struct PingHandler<TSubstream>
|
||||
where
|
||||
TSubstream: AsyncRead + AsyncWrite,
|
||||
{
|
||||
/// The actual handler which we delegate the substreams handling to.
|
||||
inner: OneShotHandler<TSubstream, protocol::Ping, protocol::Ping, protocol::PingOutput>,
|
||||
/// After a ping succeeds, how long before the next ping.
|
||||
delay_to_next_ping: Duration,
|
||||
/// When the next ping triggers.
|
||||
pub struct PingHandler<TSubstream> {
|
||||
/// Configuration options.
|
||||
config: PingConfig,
|
||||
/// The timer for when to send the next ping.
|
||||
next_ping: Delay,
|
||||
/// The connection timeout.
|
||||
connection_timeout: Duration,
|
||||
/// The current keep-alive, i.e. until when the connection that
|
||||
/// the handler operates on should be kept alive.
|
||||
connection_keepalive: KeepAlive,
|
||||
/// The pending results from inbound or outbound pings, ready
|
||||
/// to be `poll()`ed.
|
||||
pending_results: VecDeque<(Instant, PingResult)>,
|
||||
_marker: std::marker::PhantomData<TSubstream>
|
||||
}
|
||||
|
||||
impl<TSubstream> PingHandler<TSubstream>
|
||||
where
|
||||
TSubstream: AsyncRead + AsyncWrite,
|
||||
{
|
||||
/// Builds a new `PingHandler`.
|
||||
pub fn new() -> Self {
|
||||
// TODO: allow customizing timeout; depends on https://github.com/libp2p/rust-libp2p/issues/864
|
||||
/// Builds a new `PingHandler` with the given configuration.
|
||||
pub fn new(config: PingConfig) -> Self {
|
||||
let now = Instant::now();
|
||||
let connection_timeout = config.timeout * config.max_timeouts.get();
|
||||
let connection_keepalive = KeepAlive::Until(now + connection_timeout);
|
||||
let next_ping = Delay::new(now);
|
||||
PingHandler {
|
||||
inner: OneShotHandler::default(),
|
||||
next_ping: Delay::new(Instant::now()),
|
||||
delay_to_next_ping: Duration::from_secs(15),
|
||||
config,
|
||||
next_ping,
|
||||
connection_timeout,
|
||||
connection_keepalive,
|
||||
pending_results: VecDeque::with_capacity(2),
|
||||
_marker: std::marker::PhantomData
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
impl<TSubstream> Default for PingHandler<TSubstream>
|
||||
where
|
||||
TSubstream: AsyncRead + AsyncWrite,
|
||||
{
|
||||
#[inline]
|
||||
fn default() -> Self {
|
||||
PingHandler::new()
|
||||
}
|
||||
}
|
||||
|
||||
impl<TSubstream> ProtocolsHandler for PingHandler<TSubstream>
|
||||
where
|
||||
TSubstream: AsyncRead + AsyncWrite,
|
||||
{
|
||||
type InEvent = void::Void;
|
||||
type OutEvent = protocol::PingOutput;
|
||||
type InEvent = Void;
|
||||
type OutEvent = PingResult;
|
||||
type Error = ProtocolsHandlerUpgrErr<io::Error>;
|
||||
type Substream = TSubstream;
|
||||
type InboundProtocol = protocol::Ping;
|
||||
type OutboundProtocol = protocol::Ping;
|
||||
type OutboundOpenInfo = ();
|
||||
|
||||
fn listen_protocol(&self) -> Self::InboundProtocol {
|
||||
self.inner.listen_protocol()
|
||||
fn listen_protocol(&self) -> SubstreamProtocol<protocol::Ping> {
|
||||
SubstreamProtocol::new(protocol::Ping)
|
||||
}
|
||||
|
||||
fn inject_fully_negotiated_inbound(&mut self, protocol: ()) {
|
||||
self.inner.inject_fully_negotiated_inbound(protocol)
|
||||
fn inject_fully_negotiated_inbound(&mut self, _: ()) {
|
||||
// A ping from a remote peer has been answered.
|
||||
self.pending_results.push_front((Instant::now(), Ok(PingSuccess::Pong)));
|
||||
}
|
||||
|
||||
fn inject_fully_negotiated_outbound(&mut self, duration: Duration, info: Self::OutboundOpenInfo) {
|
||||
self.inner.inject_fully_negotiated_outbound(duration, info)
|
||||
fn inject_fully_negotiated_outbound(&mut self, rtt: Duration, _info: ()) {
|
||||
// A ping initiated by the local peer was answered by the remote.
|
||||
self.pending_results.push_front((Instant::now(), Ok(PingSuccess::Ping { rtt })));
|
||||
}
|
||||
|
||||
fn inject_event(&mut self, event: void::Void) {
|
||||
void::unreachable(event)
|
||||
}
|
||||
fn inject_event(&mut self, _: Void) {}
|
||||
|
||||
fn inject_dial_upgrade_error(&mut self, info: Self::OutboundOpenInfo, error: ProtocolsHandlerUpgrErr<io::Error>) {
|
||||
self.inner.inject_dial_upgrade_error(info, error)
|
||||
fn inject_dial_upgrade_error(&mut self, _info: (), error: Self::Error) {
|
||||
self.pending_results.push_front(
|
||||
(Instant::now(), Err(match error {
|
||||
ProtocolsHandlerUpgrErr::Timeout => PingFailure::Timeout,
|
||||
e => PingFailure::Other { error: Box::new(e) }
|
||||
})))
|
||||
}
|
||||
|
||||
fn connection_keep_alive(&self) -> KeepAlive {
|
||||
self.inner.connection_keep_alive()
|
||||
self.connection_keepalive
|
||||
}
|
||||
|
||||
fn poll(
|
||||
&mut self,
|
||||
) -> Poll<
|
||||
ProtocolsHandlerEvent<Self::OutboundProtocol, Self::OutboundOpenInfo, Self::OutEvent>,
|
||||
Self::Error,
|
||||
> {
|
||||
fn poll(&mut self) -> Poll<ProtocolsHandlerEvent<protocol::Ping, (), PingResult>, Self::Error> {
|
||||
if let Some((instant, result)) = self.pending_results.pop_back() {
|
||||
if result.is_ok() {
|
||||
self.connection_keepalive = KeepAlive::Until(instant + self.connection_timeout);
|
||||
let reset = match result {
|
||||
Ok(PingSuccess::Ping { .. }) => true,
|
||||
Ok(PingSuccess::Pong) => self.config.policy == PingPolicy::KeepAlive,
|
||||
_ => false
|
||||
};
|
||||
if reset {
|
||||
self.next_ping.reset(instant + self.config.interval);
|
||||
}
|
||||
}
|
||||
return Ok(Async::Ready(ProtocolsHandlerEvent::Custom(result)))
|
||||
}
|
||||
|
||||
match self.next_ping.poll() {
|
||||
Ok(Async::Ready(())) => {
|
||||
self.inner.inject_event(protocol::Ping::default());
|
||||
self.next_ping.reset(Instant::now() + Duration::from_secs(3600));
|
||||
self.next_ping.reset(Instant::now() + self.config.timeout);
|
||||
let protocol = SubstreamProtocol::new(protocol::Ping)
|
||||
.with_timeout(self.config.timeout);
|
||||
Ok(Async::Ready(ProtocolsHandlerEvent::OutboundSubstreamRequest {
|
||||
protocol,
|
||||
info: (),
|
||||
}))
|
||||
},
|
||||
Ok(Async::NotReady) => (),
|
||||
Err(_) => (),
|
||||
};
|
||||
|
||||
let event = self.inner.poll();
|
||||
if let Ok(Async::Ready(ProtocolsHandlerEvent::Custom(protocol::PingOutput::Ping(_)))) = event {
|
||||
self.next_ping.reset(Instant::now() + self.delay_to_next_ping);
|
||||
Ok(Async::NotReady) => Ok(Async::NotReady),
|
||||
Err(_) => Err(ProtocolsHandlerUpgrErr::Timer)
|
||||
}
|
||||
event
|
||||
}
|
||||
}
|
||||
|
@ -18,65 +18,74 @@
|
||||
// FROM, OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER
|
||||
// DEALINGS IN THE SOFTWARE.
|
||||
|
||||
//! Handles the `/ipfs/ping/1.0.0` protocol. This allows pinging a remote node and waiting for an
|
||||
//! answer.
|
||||
//! This module implements the `/ipfs/ping/1.0.0` protocol.
|
||||
//!
|
||||
//! The ping protocol can be used as an application-layer keep-alive functionality
|
||||
//! for connections of any [`Transport`] as well as to measure round-trip times.
|
||||
//!
|
||||
//! # Usage
|
||||
//!
|
||||
//! The `Ping` struct implements the `NetworkBehaviour` trait. When used, it will automatically
|
||||
//! send a periodic ping to nodes we are connected to. If a remote doesn't answer in time, it gets
|
||||
//! automatically disconnected.
|
||||
//! The [`Ping`] struct implements the [`NetworkBehaviour`] trait. When used with a [`Swarm`],
|
||||
//! it will respond to inbound ping requests and as necessary periodically send outbound
|
||||
//! ping requests on every established connection. If no pings are received or
|
||||
//! successfully sent within a configurable time window, [`PingHandler::connection_keep_alive`]
|
||||
//! eventually indicates to the `Swarm` that the connection should be closed.
|
||||
//!
|
||||
//! The `Ping` struct is also what handles answering to the pings sent by remotes.
|
||||
//! The `Ping` network behaviour produces [`PingEvent`]s, which may be consumed from the `Swarm`
|
||||
//! by an application, e.g. to collect statistics.
|
||||
//!
|
||||
//! When a ping succeeds, a `PingSuccess` event is generated, indicating the time the ping took.
|
||||
//! [`Swarm`]: libp2p_core::Swarm
|
||||
//! [`Transport`]: libp2p_core::Transport
|
||||
|
||||
pub mod protocol;
|
||||
pub mod handler;
|
||||
|
||||
pub use handler::{PingConfig, PingPolicy, PingResult, PingSuccess, PingFailure};
|
||||
use handler::PingHandler;
|
||||
|
||||
use futures::prelude::*;
|
||||
use libp2p_core::swarm::{ConnectedPoint, NetworkBehaviour, NetworkBehaviourAction, PollParameters};
|
||||
use libp2p_core::protocols_handler::ProtocolsHandler;
|
||||
use libp2p_core::{Multiaddr, PeerId};
|
||||
use std::{marker::PhantomData, time::Duration};
|
||||
use std::collections::VecDeque;
|
||||
use std::marker::PhantomData;
|
||||
use tokio_io::{AsyncRead, AsyncWrite};
|
||||
use void::Void;
|
||||
|
||||
/// Network behaviour that handles receiving pings sent by other nodes and periodically pings the
|
||||
/// nodes we are connected to.
|
||||
/// `Ping` is a [`NetworkBehaviour`] that responds to inbound pings and
|
||||
/// periodically sends outbound pings on every established connection.
|
||||
///
|
||||
/// See the crate root documentation for more information.
|
||||
pub struct Ping<TSubstream> {
|
||||
/// Marker to pin the generics.
|
||||
marker: PhantomData<TSubstream>,
|
||||
/// Queue of events to report to the user.
|
||||
events: Vec<PingEvent>,
|
||||
/// Configuration for outbound pings.
|
||||
config: PingConfig,
|
||||
/// Queue of events to yield to the swarm.
|
||||
events: VecDeque<PingEvent>,
|
||||
_marker: PhantomData<TSubstream>,
|
||||
}
|
||||
|
||||
/// Event generated by the `Ping` behaviour.
|
||||
pub enum PingEvent {
|
||||
/// We have successfully pinged a peer we are connected to.
|
||||
PingSuccess {
|
||||
/// Id of the peer that we pinged.
|
||||
peer: PeerId,
|
||||
/// Time elapsed between when we sent the ping and when we received the response.
|
||||
time: Duration,
|
||||
}
|
||||
/// Event generated by the `Ping` network behaviour.
|
||||
#[derive(Debug)]
|
||||
pub struct PingEvent {
|
||||
/// The peer ID of the remote.
|
||||
pub peer: PeerId,
|
||||
/// The result of an inbound or outbound ping.
|
||||
pub result: PingResult,
|
||||
}
|
||||
|
||||
impl<TSubstream> Ping<TSubstream> {
|
||||
/// Creates a `Ping`.
|
||||
pub fn new() -> Self {
|
||||
/// Creates a new `Ping` network behaviour with the given configuration.
|
||||
pub fn new(config: PingConfig) -> Self {
|
||||
Ping {
|
||||
marker: PhantomData,
|
||||
events: Vec::new(),
|
||||
config,
|
||||
events: VecDeque::new(),
|
||||
_marker: PhantomData,
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
impl<TSubstream> Default for Ping<TSubstream> {
|
||||
#[inline]
|
||||
fn default() -> Self {
|
||||
Ping::new()
|
||||
Ping::new(PingConfig::new())
|
||||
}
|
||||
}
|
||||
|
||||
@ -84,11 +93,11 @@ impl<TSubstream> NetworkBehaviour for Ping<TSubstream>
|
||||
where
|
||||
TSubstream: AsyncRead + AsyncWrite,
|
||||
{
|
||||
type ProtocolsHandler = handler::PingHandler<TSubstream>;
|
||||
type ProtocolsHandler = PingHandler<TSubstream>;
|
||||
type OutEvent = PingEvent;
|
||||
|
||||
fn new_handler(&mut self) -> Self::ProtocolsHandler {
|
||||
handler::PingHandler::default()
|
||||
PingHandler::new(self.config.clone())
|
||||
}
|
||||
|
||||
fn addresses_of_peer(&mut self, _peer_id: &PeerId) -> Vec<Multiaddr> {
|
||||
@ -99,32 +108,16 @@ where
|
||||
|
||||
fn inject_disconnected(&mut self, _: &PeerId, _: ConnectedPoint) {}
|
||||
|
||||
fn inject_node_event(
|
||||
&mut self,
|
||||
source: PeerId,
|
||||
event: <Self::ProtocolsHandler as ProtocolsHandler>::OutEvent,
|
||||
) {
|
||||
if let protocol::PingOutput::Ping(time) = event {
|
||||
self.events.push(PingEvent::PingSuccess {
|
||||
peer: source,
|
||||
time,
|
||||
})
|
||||
}
|
||||
fn inject_node_event(&mut self, peer: PeerId, result: PingResult) {
|
||||
self.events.push_front(PingEvent { peer, result })
|
||||
}
|
||||
|
||||
fn poll(
|
||||
&mut self,
|
||||
_: &mut PollParameters<'_>,
|
||||
) -> Async<
|
||||
NetworkBehaviourAction<
|
||||
<Self::ProtocolsHandler as ProtocolsHandler>::InEvent,
|
||||
Self::OutEvent,
|
||||
>,
|
||||
> {
|
||||
if !self.events.is_empty() {
|
||||
return Async::Ready(NetworkBehaviourAction::GenerateEvent(self.events.remove(0)));
|
||||
fn poll(&mut self, _: &mut PollParameters<'_>) -> Async<NetworkBehaviourAction<Void, PingEvent>>
|
||||
{
|
||||
if let Some(e) = self.events.pop_back() {
|
||||
Async::Ready(NetworkBehaviourAction::GenerateEvent(e))
|
||||
} else {
|
||||
Async::NotReady
|
||||
}
|
||||
|
||||
Async::NotReady
|
||||
}
|
||||
}
|
||||
|
@ -21,9 +21,9 @@
|
||||
use futures::{prelude::*, future, try_ready};
|
||||
use libp2p_core::{InboundUpgrade, OutboundUpgrade, UpgradeInfo, upgrade::Negotiated};
|
||||
use log::debug;
|
||||
use rand::{distributions::Standard, prelude::*, rngs::EntropyRng};
|
||||
use rand::{distributions, prelude::*};
|
||||
use std::{io, iter, time::Duration, time::Instant};
|
||||
use tokio_io::{AsyncRead, AsyncWrite};
|
||||
use tokio_io::{io as nio, AsyncRead, AsyncWrite};
|
||||
|
||||
/// Represents a prototype for an upgrade to handle the ping protocol.
|
||||
///
|
||||
@ -33,8 +33,14 @@ use tokio_io::{AsyncRead, AsyncWrite};
|
||||
/// - Listener receives the data and sends it back.
|
||||
/// - Dialer receives the data and verifies that it matches what it sent.
|
||||
///
|
||||
/// The dialer produces a `Duration`, which corresponds to the time between when we flushed the
|
||||
/// substream and when we received back the payload.
|
||||
/// The dialer produces a `Duration`, which corresponds to the round-trip time
|
||||
/// of the payload.
|
||||
///
|
||||
/// > **Note**: The round-trip time of a ping may be subject to delays induced
|
||||
/// > by the underlying transport, e.g. in the case of TCP there is
|
||||
/// > Nagle's algorithm, delayed acks and similar configuration options
|
||||
/// > which can affect latencies especially on otherwise low-volume
|
||||
/// > connections.
|
||||
#[derive(Default, Debug, Copy, Clone)]
|
||||
pub struct Ping;
|
||||
|
||||
@ -47,20 +53,33 @@ impl UpgradeInfo for Ping {
|
||||
}
|
||||
}
|
||||
|
||||
type RecvPing<T> = nio::ReadExact<Negotiated<T>, [u8; 32]>;
|
||||
type SendPong<T> = nio::WriteAll<Negotiated<T>, [u8; 32]>;
|
||||
type Flush<T> = nio::Flush<Negotiated<T>>;
|
||||
type Shutdown<T> = nio::Shutdown<Negotiated<T>>;
|
||||
|
||||
impl<TSocket> InboundUpgrade<TSocket> for Ping
|
||||
where
|
||||
TSocket: AsyncRead + AsyncWrite,
|
||||
{
|
||||
type Output = ();
|
||||
type Error = io::Error;
|
||||
type Future = future::Map<future::AndThen<future::AndThen<future::AndThen<tokio_io::io::ReadExact<Negotiated<TSocket>, [u8; 32]>, tokio_io::io::WriteAll<Negotiated<TSocket>, [u8; 32]>, fn((Negotiated<TSocket>, [u8; 32])) -> tokio_io::io::WriteAll<Negotiated<TSocket>, [u8; 32]>>, tokio_io::io::Flush<Negotiated<TSocket>>, fn((Negotiated<TSocket>, [u8; 32])) -> tokio_io::io::Flush<Negotiated<TSocket>>>, tokio_io::io::Shutdown<Negotiated<TSocket>>, fn(Negotiated<TSocket>) -> tokio_io::io::Shutdown<Negotiated<TSocket>>>, fn(Negotiated<TSocket>) -> ()>;
|
||||
type Future = future::Map<
|
||||
future::AndThen<
|
||||
future::AndThen<
|
||||
future::AndThen<
|
||||
RecvPing<TSocket>,
|
||||
SendPong<TSocket>, fn((Negotiated<TSocket>, [u8; 32])) -> SendPong<TSocket>>,
|
||||
Flush<TSocket>, fn((Negotiated<TSocket>, [u8; 32])) -> Flush<TSocket>>,
|
||||
Shutdown<TSocket>, fn(Negotiated<TSocket>) -> Shutdown<TSocket>>,
|
||||
fn(Negotiated<TSocket>) -> ()>;
|
||||
|
||||
#[inline]
|
||||
fn upgrade_inbound(self, socket: Negotiated<TSocket>, _: Self::Info) -> Self::Future {
|
||||
tokio_io::io::read_exact(socket, [0; 32])
|
||||
.and_then::<fn(_) -> _, _>(|(socket, buffer)| tokio_io::io::write_all(socket, buffer))
|
||||
.and_then::<fn(_) -> _, _>(|(socket, _)| tokio_io::io::flush(socket))
|
||||
.and_then::<fn(_) -> _, _>(|socket| tokio_io::io::shutdown(socket))
|
||||
nio::read_exact(socket, [0; 32])
|
||||
.and_then::<fn(_) -> _, _>(|(sock, buf)| nio::write_all(sock, buf))
|
||||
.and_then::<fn(_) -> _, _>(|(sock, _)| nio::flush(sock))
|
||||
.and_then::<fn(_) -> _, _>(|sock| nio::shutdown(sock))
|
||||
.map(|_| ())
|
||||
}
|
||||
}
|
||||
@ -75,143 +94,135 @@ where
|
||||
|
||||
#[inline]
|
||||
fn upgrade_outbound(self, socket: Negotiated<TSocket>, _: Self::Info) -> Self::Future {
|
||||
let payload: [u8; 32] = EntropyRng::default().sample(Standard);
|
||||
debug!("Preparing for ping with payload {:?}", payload);
|
||||
let payload: [u8; 32] = thread_rng().sample(distributions::Standard);
|
||||
debug!("Preparing ping payload {:?}", payload);
|
||||
|
||||
PingDialer {
|
||||
inner: PingDialerInner::Write {
|
||||
inner: tokio_io::io::write_all(socket, payload),
|
||||
state: PingDialerState::Write {
|
||||
inner: nio::write_all(socket, payload),
|
||||
},
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
/// Sends a ping and receives a pong.
|
||||
///
|
||||
/// Implements `Future`. Finishes when the pong has arrived and has been verified.
|
||||
/// A `PingDialer` is a future that sends a ping and expects to receive a pong.
|
||||
pub struct PingDialer<TSocket> {
|
||||
inner: PingDialerInner<TSocket>,
|
||||
state: PingDialerState<TSocket>
|
||||
}
|
||||
|
||||
enum PingDialerInner<TSocket> {
|
||||
enum PingDialerState<TSocket> {
|
||||
Write {
|
||||
inner: tokio_io::io::WriteAll<TSocket, [u8; 32]>,
|
||||
inner: nio::WriteAll<TSocket, [u8; 32]>,
|
||||
},
|
||||
Flush {
|
||||
inner: tokio_io::io::Flush<TSocket>,
|
||||
ping_payload: [u8; 32],
|
||||
inner: nio::Flush<TSocket>,
|
||||
payload: [u8; 32],
|
||||
},
|
||||
Read {
|
||||
inner: tokio_io::io::ReadExact<TSocket, [u8; 32]>,
|
||||
ping_payload: [u8; 32],
|
||||
inner: nio::ReadExact<TSocket, [u8; 32]>,
|
||||
payload: [u8; 32],
|
||||
started: Instant,
|
||||
},
|
||||
Shutdown {
|
||||
inner: tokio_io::io::Shutdown<TSocket>,
|
||||
ping_time: Duration,
|
||||
inner: nio::Shutdown<TSocket>,
|
||||
rtt: Duration,
|
||||
},
|
||||
}
|
||||
|
||||
impl<TSocket> Future for PingDialer<TSocket>
|
||||
where TSocket: AsyncRead + AsyncWrite,
|
||||
where
|
||||
TSocket: AsyncRead + AsyncWrite,
|
||||
{
|
||||
type Item = Duration;
|
||||
type Error = io::Error;
|
||||
|
||||
fn poll(&mut self) -> Poll<Self::Item, Self::Error> {
|
||||
loop {
|
||||
let new_state = match self.inner {
|
||||
PingDialerInner::Write { ref mut inner } => {
|
||||
let (socket, ping_payload) = try_ready!(inner.poll());
|
||||
PingDialerInner::Flush {
|
||||
inner: tokio_io::io::flush(socket),
|
||||
ping_payload,
|
||||
self.state = match self.state {
|
||||
PingDialerState::Write { ref mut inner } => {
|
||||
let (socket, payload) = try_ready!(inner.poll());
|
||||
PingDialerState::Flush {
|
||||
inner: nio::flush(socket),
|
||||
payload,
|
||||
}
|
||||
},
|
||||
PingDialerInner::Flush { ref mut inner, ping_payload } => {
|
||||
PingDialerState::Flush { ref mut inner, payload } => {
|
||||
let socket = try_ready!(inner.poll());
|
||||
let started = Instant::now();
|
||||
PingDialerInner::Read {
|
||||
inner: tokio_io::io::read_exact(socket, [0; 32]),
|
||||
ping_payload,
|
||||
PingDialerState::Read {
|
||||
inner: nio::read_exact(socket, [0; 32]),
|
||||
payload,
|
||||
started,
|
||||
}
|
||||
},
|
||||
PingDialerInner::Read { ref mut inner, ping_payload, started } => {
|
||||
let (socket, obtained) = try_ready!(inner.poll());
|
||||
let ping_time = started.elapsed();
|
||||
if obtained != ping_payload {
|
||||
return Err(io::Error::new(io::ErrorKind::InvalidData,
|
||||
"Received ping payload doesn't match expected"));
|
||||
PingDialerState::Read { ref mut inner, payload, started } => {
|
||||
let (socket, payload_received) = try_ready!(inner.poll());
|
||||
let rtt = started.elapsed();
|
||||
if payload_received != payload {
|
||||
return Err(io::Error::new(
|
||||
io::ErrorKind::InvalidData, "Ping payload mismatch"));
|
||||
}
|
||||
PingDialerInner::Shutdown {
|
||||
inner: tokio_io::io::shutdown(socket),
|
||||
ping_time,
|
||||
PingDialerState::Shutdown {
|
||||
inner: nio::shutdown(socket),
|
||||
rtt,
|
||||
}
|
||||
},
|
||||
PingDialerInner::Shutdown { ref mut inner, ping_time } => {
|
||||
let _ = try_ready!(inner.poll());
|
||||
return Ok(Async::Ready(ping_time));
|
||||
PingDialerState::Shutdown { ref mut inner, rtt } => {
|
||||
try_ready!(inner.poll());
|
||||
return Ok(Async::Ready(rtt));
|
||||
},
|
||||
};
|
||||
|
||||
self.inner = new_state;
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
/// Enum to merge the output of `Ping` for the dialer and listener.
|
||||
#[derive(Debug, Copy, Clone)]
|
||||
pub enum PingOutput {
|
||||
/// Received a ping and sent back a pong.
|
||||
Pong,
|
||||
/// Sent a ping and received back a pong. Contains the ping time.
|
||||
Ping(Duration),
|
||||
}
|
||||
|
||||
impl From<Duration> for PingOutput {
|
||||
#[inline]
|
||||
fn from(duration: Duration) -> PingOutput {
|
||||
PingOutput::Ping(duration)
|
||||
}
|
||||
}
|
||||
|
||||
impl From<()> for PingOutput {
|
||||
#[inline]
|
||||
fn from(_: ()) -> PingOutput {
|
||||
PingOutput::Pong
|
||||
}
|
||||
}
|
||||
|
||||
#[cfg(test)]
|
||||
mod tests {
|
||||
use tokio_tcp::{TcpListener, TcpStream};
|
||||
use super::Ping;
|
||||
use futures::{Future, Stream};
|
||||
use libp2p_core::upgrade;
|
||||
|
||||
// TODO: rewrite tests with the MemoryTransport
|
||||
use futures::prelude::*;
|
||||
use libp2p_core::{
|
||||
upgrade,
|
||||
multiaddr::multiaddr,
|
||||
transport::{
|
||||
Transport,
|
||||
ListenerEvent,
|
||||
memory::MemoryTransport
|
||||
}
|
||||
};
|
||||
use rand::{thread_rng, Rng};
|
||||
use std::time::Duration;
|
||||
|
||||
#[test]
|
||||
fn ping_pong() {
|
||||
let listener = TcpListener::bind(&"127.0.0.1:0".parse().unwrap()).unwrap();
|
||||
let listener_addr = listener.local_addr().unwrap();
|
||||
let mem_addr = multiaddr![Memory(thread_rng().gen::<u64>())];
|
||||
let mut listener = MemoryTransport.listen_on(mem_addr).unwrap();
|
||||
|
||||
let listener_addr =
|
||||
if let Ok(Async::Ready(Some(ListenerEvent::NewAddress(a)))) = listener.poll() {
|
||||
a
|
||||
} else {
|
||||
panic!("MemoryTransport not listening on an address!");
|
||||
};
|
||||
|
||||
let server = listener
|
||||
.incoming()
|
||||
.into_future()
|
||||
.map_err(|(e, _)| e)
|
||||
.and_then(|(c, _)| {
|
||||
upgrade::apply_inbound(c.unwrap(), Ping::default()).map_err(|_| panic!())
|
||||
.and_then(|(listener_event, _)| {
|
||||
let (listener_upgrade, _) = listener_event.unwrap().into_upgrade().unwrap();
|
||||
let conn = listener_upgrade.wait().unwrap();
|
||||
upgrade::apply_inbound(conn, Ping::default())
|
||||
.map_err(|e| panic!(e))
|
||||
});
|
||||
|
||||
let client = TcpStream::connect(&listener_addr)
|
||||
let client = MemoryTransport.dial(listener_addr).unwrap()
|
||||
.and_then(|c| {
|
||||
upgrade::apply_outbound(c, Ping::default()).map_err(|_| panic!())
|
||||
})
|
||||
.map(|_| ());
|
||||
upgrade::apply_outbound(c, Ping::default())
|
||||
.map_err(|e| panic!(e))
|
||||
});
|
||||
|
||||
let mut runtime = tokio::runtime::Runtime::new().unwrap();
|
||||
runtime.block_on(server.select(client).map_err(|_| panic!())).unwrap();
|
||||
runtime.spawn(server.map_err(|e| panic!(e)));
|
||||
let rtt = runtime.block_on(client).expect("RTT");
|
||||
assert!(rtt > Duration::from_secs(0));
|
||||
}
|
||||
}
|
||||
|
Reference in New Issue
Block a user