221 lines
7.1 KiB
Rust
Raw Normal View History

Gossipsub Protocol (#898) * Create gossipsub crate - Basic template, borrowed from floodsub * Add a GossipsubConfig struct and set up basic structures in the Gossipsub struct * Begin implementation of join. Adds get_random_peers helper function and adds tests * Implements gossipsub leave() * Update publishMany to incorporate gossipsub mesh and fanout logic * Use the gossipsub mesh for determining peer subscription * Remove subscribed_topics field from the Gossipsub struct * Rename gossipsubconfig to ProtocolConfig * Implement the gossipsub control messages into the Codec's Encode/Decode and modifies GossipsubRpc * Modify GossipsubActions to enums for succinctness. * Modify the memcache to store Gossipsub messages * Implement control message handling. * Update control message handling to handle multiple messages. * Handle received gossipsub messages using pre-built handlers. * Remove excess connected peer hashmap * Add extra peer mapping and consistent topic naming. * Implement heartbeat, emit_gossip and send_graft_prune. * Group logic in forwarding messages. Add messages to memcache. * Add heartbeat timer and move location of helper function. * Add gossipsub the libp2p workspace, makes layer structs public * Add logging to gossipsub - Adds the log crate and implements logging macros - Specifies versions for external crates * Add example chat for debugging purposes * Implement #868 for gossipsub. * Add rust documentation to gossipsub crate. - Adds basic documentation, overview and examples to the gossipsub crate. * Re-introduce the initial heartbeat time config. This commit also adds the inject_connected test. * Add subscribe tests. - Modifies `handle_received_subscriptions` to take a reference of subscriptions - Adds `test_subscribe` - Adds `test_handle_received_subscriptions` - Adds tests for the filter in `get_random_peers` * Add Bug fixes and further testing for gossipsub. - Corrects the tuple use of topic_hashes - Corrects JOIN logic around fanout and adding peers to the mesh - Adds test_unsubscribe - Adds test_join * Rename GossipsubMessage::msg_id -> id * Add bug fix for handling disconnected peers. * Implements (partially) #889 for Gossipsub. * handle_iwant event count tests * handle_ihave event count tests * Move layer.rs tests into separate file. * Implement clippy suggestions for gossipsub. * Modify control message tests for specific types. * Implement builder pattern for GossipsubConfig. As suggested by @twittner - The builder pattern for building GossipsubConfig struct is implemented. * Package version updates as suggested by @twittner. * Correct line lengths in gossipsub. * Correct braces in found by @twittner. * Implement @twittner's suggestions. - Uses `HashSet` where applicable - Update `FnvHashMap` to standard `HashMap` - Uses `min` function in code simplification. * Add NodeList struct to clarify topic_peers. * Cleaner handling of messagelist Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner handling of added peers. Co-Authored-By: AgeManning <Age@AgeManning.com> * handle_prune peer removed test * basic grafting tests * multiple topic grafting test * Convert &vec to slice. Co-Authored-By: AgeManning <Age@AgeManning.com> * Convert to lazy insert. Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner topic handling. Co-Authored-By: AgeManning <Age@AgeManning.com> * control pool piggybacking using HashMap.drain() in control_pool_flush going to squash this * Add Debug derives to gossipsub and correct tests. * changes from PR squash this all tests passing, but still some that need to be reconsidered test reform * Implements Arc for GossipsubRpc events * Remove support for floodsub nodes * Reconnected to disconnected peers, to mitigate timeout * Use ReadOne WriteOne with configurable max gossip sizes * Remove length delimination from RPC encoding * Prevent peer duplication in mesh * Allow oneshot handler's inactivity_timeout to be configurable * Correct peer duplication in mesh bug * Remove auto-reconnect to allow for user-level disconnects * Single long-lived inbound/outbound streams to match go implementation * Allow gossipsub topics to be optionally hashable * Improves gossipsub stream handling - Corrects the handler's keep alive. - Correct the chat example. - Instantly add peers to the mesh on subscription if the mesh is low. * Allows message validation in gossipsub * Replaces Cuckoofilter with LRUCache The false positive rate was unacceptable for rejecting messages. * Renames configuration parameter and corrects logic * Removes peer from fanout on disconnection * Add publish and fanout tests * Apply @mxinden suggestions * Resend message if outbound stream negotiated - Downgrades log warnings * Implement further reviewer suggestions - Created associated functions to avoid unnecessary cloning - Messages are rejected if their sequence numbers are not u64 - `GossipsbuConfigBuilder` has the same defaults as `GossipsubConfig` - Miscellaneous typos * Add MessageId type and remove unnecessary comments * Add a return value to propagate_message function * Adds user-customised gossipsub message ids * Adds the message id to GossipsubEvent * Implement Debug for GossipsubConfig * protocols/gossipsub: Add basic smoke test Implement a basic smoke test that: 1. Builds a fully connected graph of size N. 2. Subscribes each node to the same topic. 3. Publishes a single message. 4. Waits for all nodes to receive the above message. N and the structure of the graph are reproducibly randomized via Quickcheck. * Corrections pointed out by @mxinden * Add option to remove source id publishing * protocols/gossipsub/tests/smoke: Remove unused variable * Merge latest master * protocols/gossipsub: Move to stable futures * examples/gossipsub-chat.rs: Move to stable futures * protocols/gossipsub/src/behaviour/tests: Update to stable futures * protocols/gossipsub/tests: Update to stable futures * protocols/gossipsub: Log substream errors * protocols/gossipsub: Log outbound substream errors * Remove rust-fmt formatting * Shift to prost for protobuf compiling * Use wasm_timer for wasm compatibility Co-authored-by: Grant Wuerker <gwuerker@gmail.com> Co-authored-by: Toralf Wittner <tw@dtex.org> Co-authored-by: Pawan Dhananjay <pawandhananjay@gmail.com> Co-authored-by: Max Inden <mail@max-inden.de> Co-authored-by: Pierre Krieger <pierre.krieger1708@gmail.com>
2020-01-25 02:16:02 +11:00
// Copyright 2019 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::prelude::*;
use log::debug;
use quickcheck::{QuickCheck, TestResult};
use rand::{random, seq::SliceRandom, SeedableRng};
use std::{
io::Error,
pin::Pin,
task::{Context, Poll},
time::Duration,
};
use libp2p_core::{
identity,
multiaddr::Protocol,
muxing::StreamMuxerBox,
transport::MemoryTransport,
upgrade, Multiaddr, Transport,
Gossipsub Protocol (#898) * Create gossipsub crate - Basic template, borrowed from floodsub * Add a GossipsubConfig struct and set up basic structures in the Gossipsub struct * Begin implementation of join. Adds get_random_peers helper function and adds tests * Implements gossipsub leave() * Update publishMany to incorporate gossipsub mesh and fanout logic * Use the gossipsub mesh for determining peer subscription * Remove subscribed_topics field from the Gossipsub struct * Rename gossipsubconfig to ProtocolConfig * Implement the gossipsub control messages into the Codec's Encode/Decode and modifies GossipsubRpc * Modify GossipsubActions to enums for succinctness. * Modify the memcache to store Gossipsub messages * Implement control message handling. * Update control message handling to handle multiple messages. * Handle received gossipsub messages using pre-built handlers. * Remove excess connected peer hashmap * Add extra peer mapping and consistent topic naming. * Implement heartbeat, emit_gossip and send_graft_prune. * Group logic in forwarding messages. Add messages to memcache. * Add heartbeat timer and move location of helper function. * Add gossipsub the libp2p workspace, makes layer structs public * Add logging to gossipsub - Adds the log crate and implements logging macros - Specifies versions for external crates * Add example chat for debugging purposes * Implement #868 for gossipsub. * Add rust documentation to gossipsub crate. - Adds basic documentation, overview and examples to the gossipsub crate. * Re-introduce the initial heartbeat time config. This commit also adds the inject_connected test. * Add subscribe tests. - Modifies `handle_received_subscriptions` to take a reference of subscriptions - Adds `test_subscribe` - Adds `test_handle_received_subscriptions` - Adds tests for the filter in `get_random_peers` * Add Bug fixes and further testing for gossipsub. - Corrects the tuple use of topic_hashes - Corrects JOIN logic around fanout and adding peers to the mesh - Adds test_unsubscribe - Adds test_join * Rename GossipsubMessage::msg_id -> id * Add bug fix for handling disconnected peers. * Implements (partially) #889 for Gossipsub. * handle_iwant event count tests * handle_ihave event count tests * Move layer.rs tests into separate file. * Implement clippy suggestions for gossipsub. * Modify control message tests for specific types. * Implement builder pattern for GossipsubConfig. As suggested by @twittner - The builder pattern for building GossipsubConfig struct is implemented. * Package version updates as suggested by @twittner. * Correct line lengths in gossipsub. * Correct braces in found by @twittner. * Implement @twittner's suggestions. - Uses `HashSet` where applicable - Update `FnvHashMap` to standard `HashMap` - Uses `min` function in code simplification. * Add NodeList struct to clarify topic_peers. * Cleaner handling of messagelist Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner handling of added peers. Co-Authored-By: AgeManning <Age@AgeManning.com> * handle_prune peer removed test * basic grafting tests * multiple topic grafting test * Convert &vec to slice. Co-Authored-By: AgeManning <Age@AgeManning.com> * Convert to lazy insert. Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner topic handling. Co-Authored-By: AgeManning <Age@AgeManning.com> * control pool piggybacking using HashMap.drain() in control_pool_flush going to squash this * Add Debug derives to gossipsub and correct tests. * changes from PR squash this all tests passing, but still some that need to be reconsidered test reform * Implements Arc for GossipsubRpc events * Remove support for floodsub nodes * Reconnected to disconnected peers, to mitigate timeout * Use ReadOne WriteOne with configurable max gossip sizes * Remove length delimination from RPC encoding * Prevent peer duplication in mesh * Allow oneshot handler's inactivity_timeout to be configurable * Correct peer duplication in mesh bug * Remove auto-reconnect to allow for user-level disconnects * Single long-lived inbound/outbound streams to match go implementation * Allow gossipsub topics to be optionally hashable * Improves gossipsub stream handling - Corrects the handler's keep alive. - Correct the chat example. - Instantly add peers to the mesh on subscription if the mesh is low. * Allows message validation in gossipsub * Replaces Cuckoofilter with LRUCache The false positive rate was unacceptable for rejecting messages. * Renames configuration parameter and corrects logic * Removes peer from fanout on disconnection * Add publish and fanout tests * Apply @mxinden suggestions * Resend message if outbound stream negotiated - Downgrades log warnings * Implement further reviewer suggestions - Created associated functions to avoid unnecessary cloning - Messages are rejected if their sequence numbers are not u64 - `GossipsbuConfigBuilder` has the same defaults as `GossipsubConfig` - Miscellaneous typos * Add MessageId type and remove unnecessary comments * Add a return value to propagate_message function * Adds user-customised gossipsub message ids * Adds the message id to GossipsubEvent * Implement Debug for GossipsubConfig * protocols/gossipsub: Add basic smoke test Implement a basic smoke test that: 1. Builds a fully connected graph of size N. 2. Subscribes each node to the same topic. 3. Publishes a single message. 4. Waits for all nodes to receive the above message. N and the structure of the graph are reproducibly randomized via Quickcheck. * Corrections pointed out by @mxinden * Add option to remove source id publishing * protocols/gossipsub/tests/smoke: Remove unused variable * Merge latest master * protocols/gossipsub: Move to stable futures * examples/gossipsub-chat.rs: Move to stable futures * protocols/gossipsub/src/behaviour/tests: Update to stable futures * protocols/gossipsub/tests: Update to stable futures * protocols/gossipsub: Log substream errors * protocols/gossipsub: Log outbound substream errors * Remove rust-fmt formatting * Shift to prost for protobuf compiling * Use wasm_timer for wasm compatibility Co-authored-by: Grant Wuerker <gwuerker@gmail.com> Co-authored-by: Toralf Wittner <tw@dtex.org> Co-authored-by: Pawan Dhananjay <pawandhananjay@gmail.com> Co-authored-by: Max Inden <mail@max-inden.de> Co-authored-by: Pierre Krieger <pierre.krieger1708@gmail.com>
2020-01-25 02:16:02 +11:00
};
use libp2p_gossipsub::{Gossipsub, GossipsubConfig, GossipsubEvent, Topic};
use libp2p_plaintext::PlainText2Config;
use libp2p_swarm::Swarm;
use libp2p_yamux as yamux;
struct Graph {
pub nodes: Vec<(Multiaddr, Swarm<Gossipsub>)>,
Gossipsub Protocol (#898) * Create gossipsub crate - Basic template, borrowed from floodsub * Add a GossipsubConfig struct and set up basic structures in the Gossipsub struct * Begin implementation of join. Adds get_random_peers helper function and adds tests * Implements gossipsub leave() * Update publishMany to incorporate gossipsub mesh and fanout logic * Use the gossipsub mesh for determining peer subscription * Remove subscribed_topics field from the Gossipsub struct * Rename gossipsubconfig to ProtocolConfig * Implement the gossipsub control messages into the Codec's Encode/Decode and modifies GossipsubRpc * Modify GossipsubActions to enums for succinctness. * Modify the memcache to store Gossipsub messages * Implement control message handling. * Update control message handling to handle multiple messages. * Handle received gossipsub messages using pre-built handlers. * Remove excess connected peer hashmap * Add extra peer mapping and consistent topic naming. * Implement heartbeat, emit_gossip and send_graft_prune. * Group logic in forwarding messages. Add messages to memcache. * Add heartbeat timer and move location of helper function. * Add gossipsub the libp2p workspace, makes layer structs public * Add logging to gossipsub - Adds the log crate and implements logging macros - Specifies versions for external crates * Add example chat for debugging purposes * Implement #868 for gossipsub. * Add rust documentation to gossipsub crate. - Adds basic documentation, overview and examples to the gossipsub crate. * Re-introduce the initial heartbeat time config. This commit also adds the inject_connected test. * Add subscribe tests. - Modifies `handle_received_subscriptions` to take a reference of subscriptions - Adds `test_subscribe` - Adds `test_handle_received_subscriptions` - Adds tests for the filter in `get_random_peers` * Add Bug fixes and further testing for gossipsub. - Corrects the tuple use of topic_hashes - Corrects JOIN logic around fanout and adding peers to the mesh - Adds test_unsubscribe - Adds test_join * Rename GossipsubMessage::msg_id -> id * Add bug fix for handling disconnected peers. * Implements (partially) #889 for Gossipsub. * handle_iwant event count tests * handle_ihave event count tests * Move layer.rs tests into separate file. * Implement clippy suggestions for gossipsub. * Modify control message tests for specific types. * Implement builder pattern for GossipsubConfig. As suggested by @twittner - The builder pattern for building GossipsubConfig struct is implemented. * Package version updates as suggested by @twittner. * Correct line lengths in gossipsub. * Correct braces in found by @twittner. * Implement @twittner's suggestions. - Uses `HashSet` where applicable - Update `FnvHashMap` to standard `HashMap` - Uses `min` function in code simplification. * Add NodeList struct to clarify topic_peers. * Cleaner handling of messagelist Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner handling of added peers. Co-Authored-By: AgeManning <Age@AgeManning.com> * handle_prune peer removed test * basic grafting tests * multiple topic grafting test * Convert &vec to slice. Co-Authored-By: AgeManning <Age@AgeManning.com> * Convert to lazy insert. Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner topic handling. Co-Authored-By: AgeManning <Age@AgeManning.com> * control pool piggybacking using HashMap.drain() in control_pool_flush going to squash this * Add Debug derives to gossipsub and correct tests. * changes from PR squash this all tests passing, but still some that need to be reconsidered test reform * Implements Arc for GossipsubRpc events * Remove support for floodsub nodes * Reconnected to disconnected peers, to mitigate timeout * Use ReadOne WriteOne with configurable max gossip sizes * Remove length delimination from RPC encoding * Prevent peer duplication in mesh * Allow oneshot handler's inactivity_timeout to be configurable * Correct peer duplication in mesh bug * Remove auto-reconnect to allow for user-level disconnects * Single long-lived inbound/outbound streams to match go implementation * Allow gossipsub topics to be optionally hashable * Improves gossipsub stream handling - Corrects the handler's keep alive. - Correct the chat example. - Instantly add peers to the mesh on subscription if the mesh is low. * Allows message validation in gossipsub * Replaces Cuckoofilter with LRUCache The false positive rate was unacceptable for rejecting messages. * Renames configuration parameter and corrects logic * Removes peer from fanout on disconnection * Add publish and fanout tests * Apply @mxinden suggestions * Resend message if outbound stream negotiated - Downgrades log warnings * Implement further reviewer suggestions - Created associated functions to avoid unnecessary cloning - Messages are rejected if their sequence numbers are not u64 - `GossipsbuConfigBuilder` has the same defaults as `GossipsubConfig` - Miscellaneous typos * Add MessageId type and remove unnecessary comments * Add a return value to propagate_message function * Adds user-customised gossipsub message ids * Adds the message id to GossipsubEvent * Implement Debug for GossipsubConfig * protocols/gossipsub: Add basic smoke test Implement a basic smoke test that: 1. Builds a fully connected graph of size N. 2. Subscribes each node to the same topic. 3. Publishes a single message. 4. Waits for all nodes to receive the above message. N and the structure of the graph are reproducibly randomized via Quickcheck. * Corrections pointed out by @mxinden * Add option to remove source id publishing * protocols/gossipsub/tests/smoke: Remove unused variable * Merge latest master * protocols/gossipsub: Move to stable futures * examples/gossipsub-chat.rs: Move to stable futures * protocols/gossipsub/src/behaviour/tests: Update to stable futures * protocols/gossipsub/tests: Update to stable futures * protocols/gossipsub: Log substream errors * protocols/gossipsub: Log outbound substream errors * Remove rust-fmt formatting * Shift to prost for protobuf compiling * Use wasm_timer for wasm compatibility Co-authored-by: Grant Wuerker <gwuerker@gmail.com> Co-authored-by: Toralf Wittner <tw@dtex.org> Co-authored-by: Pawan Dhananjay <pawandhananjay@gmail.com> Co-authored-by: Max Inden <mail@max-inden.de> Co-authored-by: Pierre Krieger <pierre.krieger1708@gmail.com>
2020-01-25 02:16:02 +11:00
}
impl Future for Graph {
type Output = (Multiaddr, GossipsubEvent);
fn poll(mut self: Pin<&mut Self>, cx: &mut Context) -> Poll<Self::Output> {
for (addr, node) in &mut self.nodes {
match node.poll_next_unpin(cx) {
Poll::Ready(Some(event)) => return Poll::Ready((addr.clone(), event)),
Poll::Ready(None) => panic!("unexpected None when polling nodes"),
Poll::Pending => {}
}
}
Poll::Pending
}
}
impl Graph {
fn new_connected(num_nodes: usize, seed: u64) -> Graph {
if num_nodes == 0 {
panic!("expecting at least one node");
}
let mut rng = rand::rngs::StdRng::seed_from_u64(seed);
let mut not_connected_nodes = std::iter::once(())
.cycle()
.take(num_nodes)
.map(|_| build_node())
.collect::<Vec<(Multiaddr, Swarm<Gossipsub>)>>();
Gossipsub Protocol (#898) * Create gossipsub crate - Basic template, borrowed from floodsub * Add a GossipsubConfig struct and set up basic structures in the Gossipsub struct * Begin implementation of join. Adds get_random_peers helper function and adds tests * Implements gossipsub leave() * Update publishMany to incorporate gossipsub mesh and fanout logic * Use the gossipsub mesh for determining peer subscription * Remove subscribed_topics field from the Gossipsub struct * Rename gossipsubconfig to ProtocolConfig * Implement the gossipsub control messages into the Codec's Encode/Decode and modifies GossipsubRpc * Modify GossipsubActions to enums for succinctness. * Modify the memcache to store Gossipsub messages * Implement control message handling. * Update control message handling to handle multiple messages. * Handle received gossipsub messages using pre-built handlers. * Remove excess connected peer hashmap * Add extra peer mapping and consistent topic naming. * Implement heartbeat, emit_gossip and send_graft_prune. * Group logic in forwarding messages. Add messages to memcache. * Add heartbeat timer and move location of helper function. * Add gossipsub the libp2p workspace, makes layer structs public * Add logging to gossipsub - Adds the log crate and implements logging macros - Specifies versions for external crates * Add example chat for debugging purposes * Implement #868 for gossipsub. * Add rust documentation to gossipsub crate. - Adds basic documentation, overview and examples to the gossipsub crate. * Re-introduce the initial heartbeat time config. This commit also adds the inject_connected test. * Add subscribe tests. - Modifies `handle_received_subscriptions` to take a reference of subscriptions - Adds `test_subscribe` - Adds `test_handle_received_subscriptions` - Adds tests for the filter in `get_random_peers` * Add Bug fixes and further testing for gossipsub. - Corrects the tuple use of topic_hashes - Corrects JOIN logic around fanout and adding peers to the mesh - Adds test_unsubscribe - Adds test_join * Rename GossipsubMessage::msg_id -> id * Add bug fix for handling disconnected peers. * Implements (partially) #889 for Gossipsub. * handle_iwant event count tests * handle_ihave event count tests * Move layer.rs tests into separate file. * Implement clippy suggestions for gossipsub. * Modify control message tests for specific types. * Implement builder pattern for GossipsubConfig. As suggested by @twittner - The builder pattern for building GossipsubConfig struct is implemented. * Package version updates as suggested by @twittner. * Correct line lengths in gossipsub. * Correct braces in found by @twittner. * Implement @twittner's suggestions. - Uses `HashSet` where applicable - Update `FnvHashMap` to standard `HashMap` - Uses `min` function in code simplification. * Add NodeList struct to clarify topic_peers. * Cleaner handling of messagelist Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner handling of added peers. Co-Authored-By: AgeManning <Age@AgeManning.com> * handle_prune peer removed test * basic grafting tests * multiple topic grafting test * Convert &vec to slice. Co-Authored-By: AgeManning <Age@AgeManning.com> * Convert to lazy insert. Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner topic handling. Co-Authored-By: AgeManning <Age@AgeManning.com> * control pool piggybacking using HashMap.drain() in control_pool_flush going to squash this * Add Debug derives to gossipsub and correct tests. * changes from PR squash this all tests passing, but still some that need to be reconsidered test reform * Implements Arc for GossipsubRpc events * Remove support for floodsub nodes * Reconnected to disconnected peers, to mitigate timeout * Use ReadOne WriteOne with configurable max gossip sizes * Remove length delimination from RPC encoding * Prevent peer duplication in mesh * Allow oneshot handler's inactivity_timeout to be configurable * Correct peer duplication in mesh bug * Remove auto-reconnect to allow for user-level disconnects * Single long-lived inbound/outbound streams to match go implementation * Allow gossipsub topics to be optionally hashable * Improves gossipsub stream handling - Corrects the handler's keep alive. - Correct the chat example. - Instantly add peers to the mesh on subscription if the mesh is low. * Allows message validation in gossipsub * Replaces Cuckoofilter with LRUCache The false positive rate was unacceptable for rejecting messages. * Renames configuration parameter and corrects logic * Removes peer from fanout on disconnection * Add publish and fanout tests * Apply @mxinden suggestions * Resend message if outbound stream negotiated - Downgrades log warnings * Implement further reviewer suggestions - Created associated functions to avoid unnecessary cloning - Messages are rejected if their sequence numbers are not u64 - `GossipsbuConfigBuilder` has the same defaults as `GossipsubConfig` - Miscellaneous typos * Add MessageId type and remove unnecessary comments * Add a return value to propagate_message function * Adds user-customised gossipsub message ids * Adds the message id to GossipsubEvent * Implement Debug for GossipsubConfig * protocols/gossipsub: Add basic smoke test Implement a basic smoke test that: 1. Builds a fully connected graph of size N. 2. Subscribes each node to the same topic. 3. Publishes a single message. 4. Waits for all nodes to receive the above message. N and the structure of the graph are reproducibly randomized via Quickcheck. * Corrections pointed out by @mxinden * Add option to remove source id publishing * protocols/gossipsub/tests/smoke: Remove unused variable * Merge latest master * protocols/gossipsub: Move to stable futures * examples/gossipsub-chat.rs: Move to stable futures * protocols/gossipsub/src/behaviour/tests: Update to stable futures * protocols/gossipsub/tests: Update to stable futures * protocols/gossipsub: Log substream errors * protocols/gossipsub: Log outbound substream errors * Remove rust-fmt formatting * Shift to prost for protobuf compiling * Use wasm_timer for wasm compatibility Co-authored-by: Grant Wuerker <gwuerker@gmail.com> Co-authored-by: Toralf Wittner <tw@dtex.org> Co-authored-by: Pawan Dhananjay <pawandhananjay@gmail.com> Co-authored-by: Max Inden <mail@max-inden.de> Co-authored-by: Pierre Krieger <pierre.krieger1708@gmail.com>
2020-01-25 02:16:02 +11:00
let mut connected_nodes = vec![not_connected_nodes.pop().unwrap()];
while !not_connected_nodes.is_empty() {
connected_nodes.shuffle(&mut rng);
not_connected_nodes.shuffle(&mut rng);
let mut next = not_connected_nodes.pop().unwrap();
let connected_addr = &connected_nodes[0].0;
// Memory transport can not handle addresses with `/p2p` suffix.
let mut connected_addr_no_p2p = connected_addr.clone();
let p2p_suffix_connected = connected_addr_no_p2p.pop();
debug!(
"Connect: {} -> {}",
next.0.clone().pop().unwrap(),
p2p_suffix_connected.unwrap()
);
Swarm::dial_addr(&mut next.1, connected_addr_no_p2p).unwrap();
connected_nodes.push(next);
}
Graph {
nodes: connected_nodes,
}
}
/// Polls the graph and passes each event into the provided FnMut until it returns `true`.
fn wait_for<F>(self, mut f: F) -> Self
where
F: FnMut(GossipsubEvent) -> bool,
{
// The future below should return self. Given that it is a FnMut and not a FnOnce, one needs
// to wrap `self` in an Option, leaving a `None` behind after the final `Poll::Ready`.
let mut this = Some(self);
let fut = futures::future::poll_fn(move |cx| match &mut this {
Some(graph) => loop {
match graph.poll_unpin(cx) {
Poll::Ready((_addr, ev)) => {
if f(ev) {
return Poll::Ready(this.take().unwrap());
}
}
Poll::Pending => return Poll::Pending,
}
},
None => panic!("future called after final return"),
});
let fut = async_std::future::timeout(Duration::from_secs(10), fut);
futures::executor::block_on(fut).unwrap()
}
}
fn build_node() -> (Multiaddr, Swarm<Gossipsub>) {
Gossipsub Protocol (#898) * Create gossipsub crate - Basic template, borrowed from floodsub * Add a GossipsubConfig struct and set up basic structures in the Gossipsub struct * Begin implementation of join. Adds get_random_peers helper function and adds tests * Implements gossipsub leave() * Update publishMany to incorporate gossipsub mesh and fanout logic * Use the gossipsub mesh for determining peer subscription * Remove subscribed_topics field from the Gossipsub struct * Rename gossipsubconfig to ProtocolConfig * Implement the gossipsub control messages into the Codec's Encode/Decode and modifies GossipsubRpc * Modify GossipsubActions to enums for succinctness. * Modify the memcache to store Gossipsub messages * Implement control message handling. * Update control message handling to handle multiple messages. * Handle received gossipsub messages using pre-built handlers. * Remove excess connected peer hashmap * Add extra peer mapping and consistent topic naming. * Implement heartbeat, emit_gossip and send_graft_prune. * Group logic in forwarding messages. Add messages to memcache. * Add heartbeat timer and move location of helper function. * Add gossipsub the libp2p workspace, makes layer structs public * Add logging to gossipsub - Adds the log crate and implements logging macros - Specifies versions for external crates * Add example chat for debugging purposes * Implement #868 for gossipsub. * Add rust documentation to gossipsub crate. - Adds basic documentation, overview and examples to the gossipsub crate. * Re-introduce the initial heartbeat time config. This commit also adds the inject_connected test. * Add subscribe tests. - Modifies `handle_received_subscriptions` to take a reference of subscriptions - Adds `test_subscribe` - Adds `test_handle_received_subscriptions` - Adds tests for the filter in `get_random_peers` * Add Bug fixes and further testing for gossipsub. - Corrects the tuple use of topic_hashes - Corrects JOIN logic around fanout and adding peers to the mesh - Adds test_unsubscribe - Adds test_join * Rename GossipsubMessage::msg_id -> id * Add bug fix for handling disconnected peers. * Implements (partially) #889 for Gossipsub. * handle_iwant event count tests * handle_ihave event count tests * Move layer.rs tests into separate file. * Implement clippy suggestions for gossipsub. * Modify control message tests for specific types. * Implement builder pattern for GossipsubConfig. As suggested by @twittner - The builder pattern for building GossipsubConfig struct is implemented. * Package version updates as suggested by @twittner. * Correct line lengths in gossipsub. * Correct braces in found by @twittner. * Implement @twittner's suggestions. - Uses `HashSet` where applicable - Update `FnvHashMap` to standard `HashMap` - Uses `min` function in code simplification. * Add NodeList struct to clarify topic_peers. * Cleaner handling of messagelist Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner handling of added peers. Co-Authored-By: AgeManning <Age@AgeManning.com> * handle_prune peer removed test * basic grafting tests * multiple topic grafting test * Convert &vec to slice. Co-Authored-By: AgeManning <Age@AgeManning.com> * Convert to lazy insert. Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner topic handling. Co-Authored-By: AgeManning <Age@AgeManning.com> * control pool piggybacking using HashMap.drain() in control_pool_flush going to squash this * Add Debug derives to gossipsub and correct tests. * changes from PR squash this all tests passing, but still some that need to be reconsidered test reform * Implements Arc for GossipsubRpc events * Remove support for floodsub nodes * Reconnected to disconnected peers, to mitigate timeout * Use ReadOne WriteOne with configurable max gossip sizes * Remove length delimination from RPC encoding * Prevent peer duplication in mesh * Allow oneshot handler's inactivity_timeout to be configurable * Correct peer duplication in mesh bug * Remove auto-reconnect to allow for user-level disconnects * Single long-lived inbound/outbound streams to match go implementation * Allow gossipsub topics to be optionally hashable * Improves gossipsub stream handling - Corrects the handler's keep alive. - Correct the chat example. - Instantly add peers to the mesh on subscription if the mesh is low. * Allows message validation in gossipsub * Replaces Cuckoofilter with LRUCache The false positive rate was unacceptable for rejecting messages. * Renames configuration parameter and corrects logic * Removes peer from fanout on disconnection * Add publish and fanout tests * Apply @mxinden suggestions * Resend message if outbound stream negotiated - Downgrades log warnings * Implement further reviewer suggestions - Created associated functions to avoid unnecessary cloning - Messages are rejected if their sequence numbers are not u64 - `GossipsbuConfigBuilder` has the same defaults as `GossipsubConfig` - Miscellaneous typos * Add MessageId type and remove unnecessary comments * Add a return value to propagate_message function * Adds user-customised gossipsub message ids * Adds the message id to GossipsubEvent * Implement Debug for GossipsubConfig * protocols/gossipsub: Add basic smoke test Implement a basic smoke test that: 1. Builds a fully connected graph of size N. 2. Subscribes each node to the same topic. 3. Publishes a single message. 4. Waits for all nodes to receive the above message. N and the structure of the graph are reproducibly randomized via Quickcheck. * Corrections pointed out by @mxinden * Add option to remove source id publishing * protocols/gossipsub/tests/smoke: Remove unused variable * Merge latest master * protocols/gossipsub: Move to stable futures * examples/gossipsub-chat.rs: Move to stable futures * protocols/gossipsub/src/behaviour/tests: Update to stable futures * protocols/gossipsub/tests: Update to stable futures * protocols/gossipsub: Log substream errors * protocols/gossipsub: Log outbound substream errors * Remove rust-fmt formatting * Shift to prost for protobuf compiling * Use wasm_timer for wasm compatibility Co-authored-by: Grant Wuerker <gwuerker@gmail.com> Co-authored-by: Toralf Wittner <tw@dtex.org> Co-authored-by: Pawan Dhananjay <pawandhananjay@gmail.com> Co-authored-by: Max Inden <mail@max-inden.de> Co-authored-by: Pierre Krieger <pierre.krieger1708@gmail.com>
2020-01-25 02:16:02 +11:00
let key = identity::Keypair::generate_ed25519();
let public_key = key.public();
let transport = MemoryTransport::default()
.upgrade(upgrade::Version::V1)
.authenticate(PlainText2Config {
local_public_key: public_key.clone(),
})
.multiplex(yamux::Config::default())
.map(|(p, m), _| (p, StreamMuxerBox::new(m)))
.map_err(|e| -> Error { panic!("Failed to create transport: {:?}", e) })
Gossipsub Protocol (#898) * Create gossipsub crate - Basic template, borrowed from floodsub * Add a GossipsubConfig struct and set up basic structures in the Gossipsub struct * Begin implementation of join. Adds get_random_peers helper function and adds tests * Implements gossipsub leave() * Update publishMany to incorporate gossipsub mesh and fanout logic * Use the gossipsub mesh for determining peer subscription * Remove subscribed_topics field from the Gossipsub struct * Rename gossipsubconfig to ProtocolConfig * Implement the gossipsub control messages into the Codec's Encode/Decode and modifies GossipsubRpc * Modify GossipsubActions to enums for succinctness. * Modify the memcache to store Gossipsub messages * Implement control message handling. * Update control message handling to handle multiple messages. * Handle received gossipsub messages using pre-built handlers. * Remove excess connected peer hashmap * Add extra peer mapping and consistent topic naming. * Implement heartbeat, emit_gossip and send_graft_prune. * Group logic in forwarding messages. Add messages to memcache. * Add heartbeat timer and move location of helper function. * Add gossipsub the libp2p workspace, makes layer structs public * Add logging to gossipsub - Adds the log crate and implements logging macros - Specifies versions for external crates * Add example chat for debugging purposes * Implement #868 for gossipsub. * Add rust documentation to gossipsub crate. - Adds basic documentation, overview and examples to the gossipsub crate. * Re-introduce the initial heartbeat time config. This commit also adds the inject_connected test. * Add subscribe tests. - Modifies `handle_received_subscriptions` to take a reference of subscriptions - Adds `test_subscribe` - Adds `test_handle_received_subscriptions` - Adds tests for the filter in `get_random_peers` * Add Bug fixes and further testing for gossipsub. - Corrects the tuple use of topic_hashes - Corrects JOIN logic around fanout and adding peers to the mesh - Adds test_unsubscribe - Adds test_join * Rename GossipsubMessage::msg_id -> id * Add bug fix for handling disconnected peers. * Implements (partially) #889 for Gossipsub. * handle_iwant event count tests * handle_ihave event count tests * Move layer.rs tests into separate file. * Implement clippy suggestions for gossipsub. * Modify control message tests for specific types. * Implement builder pattern for GossipsubConfig. As suggested by @twittner - The builder pattern for building GossipsubConfig struct is implemented. * Package version updates as suggested by @twittner. * Correct line lengths in gossipsub. * Correct braces in found by @twittner. * Implement @twittner's suggestions. - Uses `HashSet` where applicable - Update `FnvHashMap` to standard `HashMap` - Uses `min` function in code simplification. * Add NodeList struct to clarify topic_peers. * Cleaner handling of messagelist Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner handling of added peers. Co-Authored-By: AgeManning <Age@AgeManning.com> * handle_prune peer removed test * basic grafting tests * multiple topic grafting test * Convert &vec to slice. Co-Authored-By: AgeManning <Age@AgeManning.com> * Convert to lazy insert. Co-Authored-By: AgeManning <Age@AgeManning.com> * Cleaner topic handling. Co-Authored-By: AgeManning <Age@AgeManning.com> * control pool piggybacking using HashMap.drain() in control_pool_flush going to squash this * Add Debug derives to gossipsub and correct tests. * changes from PR squash this all tests passing, but still some that need to be reconsidered test reform * Implements Arc for GossipsubRpc events * Remove support for floodsub nodes * Reconnected to disconnected peers, to mitigate timeout * Use ReadOne WriteOne with configurable max gossip sizes * Remove length delimination from RPC encoding * Prevent peer duplication in mesh * Allow oneshot handler's inactivity_timeout to be configurable * Correct peer duplication in mesh bug * Remove auto-reconnect to allow for user-level disconnects * Single long-lived inbound/outbound streams to match go implementation * Allow gossipsub topics to be optionally hashable * Improves gossipsub stream handling - Corrects the handler's keep alive. - Correct the chat example. - Instantly add peers to the mesh on subscription if the mesh is low. * Allows message validation in gossipsub * Replaces Cuckoofilter with LRUCache The false positive rate was unacceptable for rejecting messages. * Renames configuration parameter and corrects logic * Removes peer from fanout on disconnection * Add publish and fanout tests * Apply @mxinden suggestions * Resend message if outbound stream negotiated - Downgrades log warnings * Implement further reviewer suggestions - Created associated functions to avoid unnecessary cloning - Messages are rejected if their sequence numbers are not u64 - `GossipsbuConfigBuilder` has the same defaults as `GossipsubConfig` - Miscellaneous typos * Add MessageId type and remove unnecessary comments * Add a return value to propagate_message function * Adds user-customised gossipsub message ids * Adds the message id to GossipsubEvent * Implement Debug for GossipsubConfig * protocols/gossipsub: Add basic smoke test Implement a basic smoke test that: 1. Builds a fully connected graph of size N. 2. Subscribes each node to the same topic. 3. Publishes a single message. 4. Waits for all nodes to receive the above message. N and the structure of the graph are reproducibly randomized via Quickcheck. * Corrections pointed out by @mxinden * Add option to remove source id publishing * protocols/gossipsub/tests/smoke: Remove unused variable * Merge latest master * protocols/gossipsub: Move to stable futures * examples/gossipsub-chat.rs: Move to stable futures * protocols/gossipsub/src/behaviour/tests: Update to stable futures * protocols/gossipsub/tests: Update to stable futures * protocols/gossipsub: Log substream errors * protocols/gossipsub: Log outbound substream errors * Remove rust-fmt formatting * Shift to prost for protobuf compiling * Use wasm_timer for wasm compatibility Co-authored-by: Grant Wuerker <gwuerker@gmail.com> Co-authored-by: Toralf Wittner <tw@dtex.org> Co-authored-by: Pawan Dhananjay <pawandhananjay@gmail.com> Co-authored-by: Max Inden <mail@max-inden.de> Co-authored-by: Pierre Krieger <pierre.krieger1708@gmail.com>
2020-01-25 02:16:02 +11:00
.boxed();
let peer_id = public_key.clone().into_peer_id();
let behaviour = Gossipsub::new(peer_id.clone(), GossipsubConfig::default());
let mut swarm = Swarm::new(transport, behaviour, peer_id);
let port = 1 + random::<u64>();
let mut addr: Multiaddr = Protocol::Memory(port).into();
Swarm::listen_on(&mut swarm, addr.clone()).unwrap();
addr = addr.with(libp2p_core::multiaddr::Protocol::P2p(
public_key.into_peer_id().into(),
));
(addr, swarm)
}
#[test]
fn multi_hop_propagation() {
let _ = env_logger::try_init();
fn prop(num_nodes: usize, seed: u64) -> TestResult {
if num_nodes < 2 || num_nodes > 100 {
return TestResult::discard();
}
debug!("number nodes: {:?}, seed: {:?}", num_nodes, seed);
let mut graph = Graph::new_connected(num_nodes, seed);
let number_nodes = graph.nodes.len();
// Subscribe each node to the same topic.
let topic = Topic::new("test-net".into());
for (_addr, node) in &mut graph.nodes {
node.subscribe(topic.clone());
}
// Wait for all nodes to be subscribed.
let mut subscribed = 0;
graph = graph.wait_for(move |ev| {
if let GossipsubEvent::Subscribed { .. } = ev {
subscribed += 1;
if subscribed == (number_nodes - 1) * 2 {
return true;
}
}
false
});
// Publish a single message.
graph.nodes[0].1.publish(&topic, vec![1, 2, 3]);
// Wait for all nodes to receive the published message.
let mut received_msgs = 0;
graph.wait_for(move |ev| {
if let GossipsubEvent::Message(..) = ev {
received_msgs += 1;
if received_msgs == number_nodes - 1 {
return true;
}
}
false
});
TestResult::passed()
}
QuickCheck::new()
.max_tests(10)
.quickcheck(prop as fn(usize, u64) -> TestResult)
}