Rename DistAlgorithm to ConsensusProtocol.

This commit is contained in:
Andreas Fackler 2019-01-08 09:30:37 +01:00 committed by Andreas Fackler
parent a71611d8f2
commit 87b1d45d97
27 changed files with 174 additions and 174 deletions

View File

@ -46,7 +46,7 @@ use crate::network::messaging::Messaging;
use crate::network::{commst, connection};
use hbbft::broadcast::{Broadcast, Message};
use hbbft::crypto::{poly::Poly, SecretKey, SecretKeySet};
use hbbft::{DistAlgorithm, NetworkInfo, SourcedMessage};
use hbbft::{ConsensusProtocol, NetworkInfo, SourcedMessage};
/// This is a structure to start a consensus node.
pub struct Node<T> {

View File

@ -15,7 +15,7 @@ use signifix::{metric, TryFrom};
use hbbft::dynamic_honey_badger::DynamicHoneyBadger;
use hbbft::queueing_honey_badger::{Batch, QueueingHoneyBadger};
use hbbft::sender_queue::{Message, SenderQueue};
use hbbft::{DaStep, DistAlgorithm, NetworkInfo, Step, Target};
use hbbft::{ConsensusProtocol, CpStep, NetworkInfo, Step, Target};
const VERSION: &str = env!("CARGO_PKG_VERSION");
const USAGE: &str = "
@ -60,14 +60,14 @@ type Transaction = Vec<u8>;
/// A serialized message with a sender and the timestamp of arrival.
#[derive(Eq, PartialEq, Debug)]
struct TimestampedMessage<D: DistAlgorithm> {
struct TimestampedMessage<D: ConsensusProtocol> {
time: Duration,
sender_id: D::NodeId,
target: Target<D::NodeId>,
message: Vec<u8>,
}
impl<D: DistAlgorithm> Clone for TimestampedMessage<D>
impl<D: ConsensusProtocol> Clone for TimestampedMessage<D>
where
D::Message: Clone,
{
@ -95,7 +95,7 @@ pub struct HwQuality {
}
/// A "node" running an instance of the algorithm `D`.
pub struct TestNode<D: DistAlgorithm> {
pub struct TestNode<D: ConsensusProtocol> {
/// This node's own ID.
id: D::NodeId,
/// The instance of the broadcast algorithm.
@ -118,14 +118,14 @@ pub struct TestNode<D: DistAlgorithm> {
hw_quality: HwQuality,
}
type TestNodeStepResult<D> = DaStep<D>;
type TestNodeStepResult<D> = CpStep<D>;
impl<D: DistAlgorithm> TestNode<D>
impl<D: ConsensusProtocol> TestNode<D>
where
D::Message: Serialize + DeserializeOwned,
{
/// Creates a new test node with the given broadcast instance.
fn new((algo, step): (D, DaStep<D>), hw_quality: HwQuality) -> TestNode<D> {
fn new((algo, step): (D, CpStep<D>), hw_quality: HwQuality) -> TestNode<D> {
let out_queue = step
.messages
.into_iter()
@ -231,11 +231,11 @@ where
}
/// A collection of `TestNode`s representing a network.
pub struct TestNetwork<D: DistAlgorithm> {
pub struct TestNetwork<D: ConsensusProtocol> {
nodes: BTreeMap<D::NodeId, TestNode<D>>,
}
impl<D: DistAlgorithm<NodeId = NodeId>> TestNetwork<D>
impl<D: ConsensusProtocol<NodeId = NodeId>> TestNetwork<D>
where
D::Message: Serialize + DeserializeOwned + Clone,
{
@ -248,7 +248,7 @@ where
rng: &mut R,
) -> TestNetwork<D>
where
F: Fn(NetworkInfo<NodeId>, &mut R) -> (D, DaStep<D>),
F: Fn(NetworkInfo<NodeId>, &mut R) -> (D, CpStep<D>),
{
let node_ids = (0..(good_num + adv_num)).map(NodeId);
let netinfos =

View File

@ -13,7 +13,7 @@ use super::sbv_broadcast::{self, Message as SbvMessage, SbvBroadcast};
use super::{Error, FaultKind, Message, MessageContent, Result, Step};
use crate::fault_log::Fault;
use crate::threshold_sign::{self, Message as TsMessage, ThresholdSign};
use crate::{DistAlgorithm, NetworkInfo, NodeIdT, SessionIdT, Target};
use crate::{ConsensusProtocol, NetworkInfo, NodeIdT, SessionIdT, Target};
/// The state of the current epoch's coin. In some epochs this is fixed, in others it starts
/// with in `InProgress`.
@ -158,7 +158,7 @@ pub struct BinaryAgreement<N, S> {
/// The estimate of the decision value in the current epoch.
estimated: Option<bool>,
/// A permanent, latching copy of the output value. This copy is required because `output` can
/// be consumed using `DistAlgorithm::next_output` immediately after the instance finishing to
/// be consumed using `ConsensusProtocol::next_output` immediately after the instance finishing to
/// handle a message, in which case it would otherwise be unknown whether the output value was
/// ever there at all. While the output value will still be required in a later epoch to decide
/// the termination state.
@ -171,7 +171,7 @@ pub struct BinaryAgreement<N, S> {
coin_state: CoinState<N>,
}
impl<N: NodeIdT, S: SessionIdT> DistAlgorithm for BinaryAgreement<N, S> {
impl<N: NodeIdT, S: SessionIdT> ConsensusProtocol for BinaryAgreement<N, S> {
type NodeId = N;
type Input = bool;
type Output = bool;

View File

@ -13,7 +13,7 @@ use super::merkle::{Digest, MerkleTree, Proof};
use super::message::HexProof;
use super::{Error, FaultKind, Message, Result};
use crate::fault_log::Fault;
use crate::{DistAlgorithm, NetworkInfo, NodeIdT, Target};
use crate::{ConsensusProtocol, NetworkInfo, NodeIdT, Target};
type RseResult<T> = result::Result<T, rse::Error>;
@ -41,9 +41,9 @@ pub struct Broadcast<N> {
}
/// A `Broadcast` step, containing at most one output.
pub type Step<N> = crate::DaStep<Broadcast<N>>;
pub type Step<N> = crate::CpStep<Broadcast<N>>;
impl<N: NodeIdT> DistAlgorithm for Broadcast<N> {
impl<N: NodeIdT> ConsensusProtocol for Broadcast<N> {
type NodeId = N;
type Input = Vec<u8>;
type Output = Self::Input;

View File

@ -20,7 +20,7 @@ use crate::honey_badger::{self, HoneyBadger, Message as HbMessage};
use crate::sync_key_gen::{Ack, AckOutcome, Part, PartOutcome, SyncKeyGen};
use crate::util;
use crate::{Contribution, DistAlgorithm, Epoched, NetworkInfo, NodeIdT, Target};
use crate::{ConsensusProtocol, Contribution, Epoched, NetworkInfo, NodeIdT, Target};
/// A Honey Badger instance that can handle adding and removing nodes.
#[derive(Derivative)]
@ -42,7 +42,7 @@ pub struct DynamicHoneyBadger<C, N: Ord> {
pub(super) key_gen_state: Option<KeyGenState<N>>,
}
impl<C, N> DistAlgorithm for DynamicHoneyBadger<C, N>
impl<C, N> ConsensusProtocol for DynamicHoneyBadger<C, N>
where
C: Contribution + Serialize + DeserializeOwned,
N: NodeIdT + Serialize + DeserializeOwned,

View File

@ -89,7 +89,7 @@ pub use self::dynamic_honey_badger::DynamicHoneyBadger;
pub use self::error::{Error, FaultKind, Result};
/// A `DynamicHoneyBadger` step, possibly containing multiple outputs.
pub type Step<C, N> = crate::DaStep<DynamicHoneyBadger<C, N>>;
pub type Step<C, N> = crate::CpStep<DynamicHoneyBadger<C, N>>;
/// The user input for `DynamicHoneyBadger`.
#[derive(Clone, Debug)]

View File

@ -2,7 +2,7 @@
//! algorithm.
//!
//! Each algorithm can propogate their faulty node logs upwards to a calling algorithm via
//! `DistAlgorihm`'s `.handle_input()` and `.handle_message()` trait methods.
//! `ConsensusProtocol`'s `.handle_input()` and `.handle_message()` trait methods.
pub use failure::Fail;

View File

@ -9,7 +9,7 @@ use serde_derive::{Deserialize, Serialize};
use super::epoch_state::EpochState;
use super::{Batch, Error, FaultKind, HoneyBadgerBuilder, Message, Result};
use crate::{Contribution, DistAlgorithm, Fault, NetworkInfo, NodeIdT};
use crate::{ConsensusProtocol, Contribution, Fault, NetworkInfo, NodeIdT};
use super::Params;
@ -33,9 +33,9 @@ pub struct HoneyBadger<C, N> {
}
/// A `HoneyBadger` step, possibly containing multiple outputs.
pub type Step<C, N> = crate::DaStep<HoneyBadger<C, N>>;
pub type Step<C, N> = crate::CpStep<HoneyBadger<C, N>>;
impl<C, N> DistAlgorithm for HoneyBadger<C, N>
impl<C, N> ConsensusProtocol for HoneyBadger<C, N>
where
C: Contribution + Serialize + DeserializeOwned,
N: NodeIdT,

View File

@ -144,5 +144,5 @@ pub use crate::fault_log::{Fault, FaultLog};
pub use crate::messaging::{SourcedMessage, Target, TargetedMessage};
pub use crate::network_info::NetworkInfo;
pub use crate::traits::{
Contribution, DaStep, DistAlgorithm, Epoched, Message, NodeIdT, SessionIdT, Step,
ConsensusProtocol, Contribution, CpStep, Epoched, Message, NodeIdT, SessionIdT, Step,
};

View File

@ -36,7 +36,7 @@ use crate::dynamic_honey_badger::{
self, Batch as DhbBatch, DynamicHoneyBadger, FaultKind, JoinPlan, Message, Step as DhbStep,
};
use crate::transaction_queue::TransactionQueue;
use crate::{Contribution, DistAlgorithm, NetworkInfo, NodeIdT};
use crate::{ConsensusProtocol, Contribution, NetworkInfo, NodeIdT};
pub use crate::dynamic_honey_badger::{Change, ChangeState, Input};
@ -164,7 +164,7 @@ pub struct QueueingHoneyBadger<T, N: Ord, Q> {
/// A `QueueingHoneyBadger` step, possibly containing multiple outputs.
pub type Step<T, N> = crate::Step<Message<N>, Batch<T, N>, N, FaultKind>;
impl<T, N, Q> DistAlgorithm for QueueingHoneyBadger<T, N, Q>
impl<T, N, Q> ConsensusProtocol for QueueingHoneyBadger<T, N, Q>
where
T: Contribution + Serialize + DeserializeOwned + Clone,
N: NodeIdT + Serialize + DeserializeOwned,

View File

@ -8,10 +8,10 @@ use rand::Rng;
use serde::{de::DeserializeOwned, Serialize};
use super::{
Error, Message, SenderQueue, SenderQueueableDistAlgorithm, SenderQueueableMessage,
Error, Message, SenderQueue, SenderQueueableConsensusProtocol, SenderQueueableMessage,
SenderQueueableOutput,
};
use crate::{Contribution, DaStep, NodeIdT};
use crate::{Contribution, CpStep, NodeIdT};
use crate::dynamic_honey_badger::{
Batch, Change, ChangeState, DynamicHoneyBadger, Error as DhbError, JoinPlan,
@ -76,7 +76,7 @@ impl<N: Ord> SenderQueueableMessage for DhbMessage<N> {
}
}
impl<C, N> SenderQueueableDistAlgorithm for DynamicHoneyBadger<C, N>
impl<C, N> SenderQueueableConsensusProtocol for DynamicHoneyBadger<C, N>
where
C: Contribution + Serialize + DeserializeOwned,
N: NodeIdT + Serialize + DeserializeOwned,
@ -86,7 +86,7 @@ where
}
}
type Result<C, N> = result::Result<DaStep<SenderQueue<DynamicHoneyBadger<C, N>>>, Error<DhbError>>;
type Result<C, N> = result::Result<CpStep<SenderQueue<DynamicHoneyBadger<C, N>>>, Error<DhbError>>;
impl<C, N> SenderQueue<DynamicHoneyBadger<C, N>>
where

View File

@ -2,7 +2,7 @@ use std::collections::BTreeSet;
use serde::{de::DeserializeOwned, Serialize};
use super::{SenderQueueableDistAlgorithm, SenderQueueableMessage, SenderQueueableOutput};
use super::{SenderQueueableConsensusProtocol, SenderQueueableMessage, SenderQueueableOutput};
use crate::honey_badger::{Batch, HoneyBadger, Message};
use crate::{Contribution, Epoched, NodeIdT};
@ -48,7 +48,7 @@ where
}
}
impl<C, N> SenderQueueableDistAlgorithm for HoneyBadger<C, N>
impl<C, N> SenderQueueableConsensusProtocol for HoneyBadger<C, N>
where
C: Contribution + Serialize + DeserializeOwned,
N: NodeIdT,

View File

@ -1,6 +1,6 @@
//! # Sender queue
//!
//! A sender queue allows a `DistAlgorithm` that outputs `Epoched` messages to buffer those outgoing
//! A sender queue allows a `ConsensusProtocol` that outputs `Epoched` messages to buffer those outgoing
//! messages based on their epochs. A message is sent to its recipient only when the recipient's
//! epoch matches the epoch of the message. Thus no queueing is required for incoming messages since
//! any incoming messages with non-matching epochs can be safely discarded.
@ -18,7 +18,7 @@ use std::fmt::Debug;
use log::debug;
use crate::traits::EpochT;
use crate::{DaStep, DistAlgorithm, Epoched, NodeIdT, Target};
use crate::{ConsensusProtocol, CpStep, Epoched, NodeIdT, Target};
pub use self::error::Error;
pub use self::message::Message;
@ -61,20 +61,20 @@ where
fn output_epoch(&self) -> E;
}
/// A `DistAlgorithm` that can be wrapped by a sender queue.
pub trait SenderQueueableDistAlgorithm: Epoched + DistAlgorithm {
/// The maximum number of subsequent future epochs that the `DistAlgorithm` is allowed to handle
/// A `ConsensusProtocol` that can be wrapped by a sender queue.
pub trait SenderQueueableConsensusProtocol: Epoched + ConsensusProtocol {
/// The maximum number of subsequent future epochs that the `ConsensusProtocol` is allowed to handle
/// messages for.
fn max_future_epochs(&self) -> u64;
}
/// A map with outgoing messages, per epoch and per target node.
pub type OutgoingQueue<D> = BTreeMap<
<D as DistAlgorithm>::NodeId,
BTreeMap<<D as Epoched>::Epoch, Vec<<D as DistAlgorithm>::Message>>,
<D as ConsensusProtocol>::NodeId,
BTreeMap<<D as Epoched>::Epoch, Vec<<D as ConsensusProtocol>::Message>>,
>;
/// An instance of `DistAlgorithm` wrapped with a queue of outgoing messages, that is, a sender
/// An instance of `ConsensusProtocol` wrapped with a queue of outgoing messages, that is, a sender
/// queue. This wrapping ensures that the messages sent to remote instances lead to progress of the
/// entire consensus network. In particular, messages to lagging remote nodes are queued and sent
/// only when those nodes' epochs match the queued messages' epochs. Thus all nodes can handle
@ -83,9 +83,9 @@ pub type OutgoingQueue<D> = BTreeMap<
#[derive(Debug)]
pub struct SenderQueue<D>
where
D: SenderQueueableDistAlgorithm,
D: SenderQueueableConsensusProtocol,
{
/// The managed `DistAlgorithm` instance.
/// The managed `ConsensusProtocol` instance.
algo: D,
/// Our node ID.
our_id: D::NodeId,
@ -114,11 +114,11 @@ where
}
/// A `SenderQueue` step. The output corresponds to the wrapped algorithm.
pub type Step<D> = crate::DaStep<SenderQueue<D>>;
pub type Step<D> = crate::CpStep<SenderQueue<D>>;
impl<D> DistAlgorithm for SenderQueue<D>
impl<D> ConsensusProtocol for SenderQueue<D>
where
D: SenderQueueableDistAlgorithm + Debug,
D: SenderQueueableConsensusProtocol + Debug,
D::Message: Clone + SenderQueueableMessage<Epoch = D::Epoch>,
D::NodeId: NodeIdT,
D::Output: SenderQueueableOutput<D::NodeId, D::Epoch>,
@ -134,7 +134,7 @@ where
&mut self,
input: Self::Input,
rng: &mut R,
) -> Result<DaStep<Self>, Error<D::Error>> {
) -> Result<CpStep<Self>, Error<D::Error>> {
self.handle_input(input, rng)
}
@ -143,7 +143,7 @@ where
sender_id: &D::NodeId,
message: Self::Message,
rng: &mut R,
) -> Result<DaStep<Self>, Error<D::Error>> {
) -> Result<CpStep<Self>, Error<D::Error>> {
self.handle_message(sender_id, message, rng)
}
@ -158,7 +158,7 @@ where
impl<D> SenderQueue<D>
where
D: SenderQueueableDistAlgorithm + Debug,
D: SenderQueueableConsensusProtocol + Debug,
D::Message: Clone + SenderQueueableMessage<Epoch = D::Epoch>,
D::NodeId: NodeIdT,
D::Output: SenderQueueableOutput<D::NodeId, D::Epoch>,
@ -177,7 +177,7 @@ where
&mut self,
input: D::Input,
rng: &mut R,
) -> Result<DaStep<Self>, Error<D::Error>> {
) -> Result<CpStep<Self>, Error<D::Error>> {
if self.is_removed {
return Ok(Step::<D>::default());
}
@ -192,7 +192,7 @@ where
sender_id: &D::NodeId,
message: Message<D::Message>,
rng: &mut R,
) -> Result<DaStep<Self>, Error<D::Error>> {
) -> Result<CpStep<Self>, Error<D::Error>> {
if self.is_removed {
return Ok(Step::<D>::default());
}
@ -214,9 +214,9 @@ where
/// Applies `f` to the wrapped algorithm and converts the step in the result to a sender queue
/// step, deferring or dropping messages, where necessary.
fn apply<F>(&mut self, f: F) -> Result<DaStep<Self>, Error<D::Error>>
fn apply<F>(&mut self, f: F) -> Result<CpStep<Self>, Error<D::Error>>
where
F: FnOnce(&mut D) -> Result<DaStep<D>, D::Error>,
F: FnOnce(&mut D) -> Result<CpStep<D>, D::Error>,
{
let mut step = f(&mut self.algo).map_err(Error::Apply)?;
let mut sender_queue_step = self.update_epoch(&step);
@ -226,7 +226,7 @@ where
}
/// Handles an epoch start announcement.
fn handle_epoch_started(&mut self, sender_id: &D::NodeId, epoch: D::Epoch) -> DaStep<Self> {
fn handle_epoch_started(&mut self, sender_id: &D::NodeId, epoch: D::Epoch) -> CpStep<Self> {
self.peer_epochs
.entry(sender_id.clone())
.and_modify(|e| {
@ -243,9 +243,9 @@ where
}
/// Processes an announcement of a new epoch update received from a remote node.
fn process_new_epoch(&mut self, sender_id: &D::NodeId, epoch: D::Epoch) -> DaStep<Self> {
fn process_new_epoch(&mut self, sender_id: &D::NodeId, epoch: D::Epoch) -> CpStep<Self> {
let queue = match self.outgoing_queue.get_mut(sender_id) {
None => return DaStep::<Self>::default(),
None => return CpStep::<Self>::default(),
Some(queue) => queue,
};
let earlier_keys: Vec<_> = queue
@ -268,12 +268,12 @@ where
sender_id: &D::NodeId,
content: D::Message,
rng: &mut R,
) -> Result<DaStep<Self>, Error<D::Error>> {
) -> Result<CpStep<Self>, Error<D::Error>> {
self.apply(|algo| algo.handle_message(sender_id, content, rng))
}
/// Updates the current Honey Badger epoch.
fn update_epoch(&mut self, step: &DaStep<D>) -> DaStep<Self> {
fn update_epoch(&mut self, step: &CpStep<D>) -> CpStep<Self> {
if step.output.is_empty() {
return Step::<D>::default();
}
@ -327,7 +327,7 @@ where
/// decomposing a `Target::All` message into `Target::Node` messages and sending some of the
/// resulting messages while placing onto the queue those remaining messages whose recipient is
/// currently at an earlier epoch.
fn defer_messages(&mut self, step: &mut DaStep<D>) {
fn defer_messages(&mut self, step: &mut CpStep<D>) {
let max_future_epochs = self.algo.max_future_epochs();
// Append the deferred messages onto the queues.
for (id, message) in step.defer_messages(&self.peer_epochs, max_future_epochs) {
@ -397,7 +397,7 @@ where
/// instance of `SenderQueue`.
pub struct SenderQueueBuilder<D>
where
D: SenderQueueableDistAlgorithm,
D: SenderQueueableConsensusProtocol,
{
algo: D,
peer_epochs: BTreeMap<D::NodeId, D::Epoch>,
@ -405,7 +405,7 @@ where
impl<D> SenderQueueBuilder<D>
where
D: SenderQueueableDistAlgorithm + Debug,
D: SenderQueueableConsensusProtocol + Debug,
D::Message: Clone + SenderQueueableMessage<Epoch = D::Epoch>,
D::NodeId: NodeIdT,
D::Output: SenderQueueableOutput<D::NodeId, D::Epoch>,
@ -428,7 +428,7 @@ where
}
/// Creates a new sender queue and returns the `Step` with the initial message.
pub fn build(self, our_id: D::NodeId) -> (SenderQueue<D>, DaStep<SenderQueue<D>>) {
pub fn build(self, our_id: D::NodeId) -> (SenderQueue<D>, CpStep<SenderQueue<D>>) {
let epoch = self.algo.epoch();
let sq = SenderQueue {
algo: self.algo,

View File

@ -7,10 +7,10 @@ use rand::distributions::{Distribution, Standard};
use rand::Rng;
use serde::{de::DeserializeOwned, Serialize};
use super::{Error, SenderQueue, SenderQueueableDistAlgorithm};
use super::{Error, SenderQueue, SenderQueueableConsensusProtocol};
use crate::queueing_honey_badger::{Change, Error as QhbError, QueueingHoneyBadger};
use crate::transaction_queue::TransactionQueue;
use crate::{Contribution, DaStep, Epoched, NodeIdT};
use crate::{Contribution, CpStep, Epoched, NodeIdT};
impl<T, N, Q> Epoched for QueueingHoneyBadger<T, N, Q>
where
@ -26,7 +26,7 @@ where
}
}
impl<T, N, Q> SenderQueueableDistAlgorithm for QueueingHoneyBadger<T, N, Q>
impl<T, N, Q> SenderQueueableConsensusProtocol for QueueingHoneyBadger<T, N, Q>
where
T: Contribution + Serialize + DeserializeOwned + Clone,
N: NodeIdT + Serialize + DeserializeOwned,
@ -39,7 +39,7 @@ where
}
type Result<T, N, Q> =
result::Result<DaStep<SenderQueue<QueueingHoneyBadger<T, N, Q>>>, Error<QhbError>>;
result::Result<CpStep<SenderQueue<QueueingHoneyBadger<T, N, Q>>>, Error<QhbError>>;
impl<T, N, Q> SenderQueue<QueueingHoneyBadger<T, N, Q>>
where

View File

@ -9,7 +9,7 @@ use serde_derive::Serialize;
use super::proposal_state::{ProposalState, Step as ProposalStep};
use super::{Error, FaultKind, Message, MessageContent, Result};
use crate::{util, DistAlgorithm, NetworkInfo, NodeIdT, SessionIdT};
use crate::{util, ConsensusProtocol, NetworkInfo, NodeIdT, SessionIdT};
use rand::Rng;
/// A `Subset` step, possibly containing several outputs.
@ -41,7 +41,7 @@ pub struct Subset<N, S> {
decided: bool,
}
impl<N: NodeIdT, S: SessionIdT> DistAlgorithm for Subset<N, S> {
impl<N: NodeIdT, S: SessionIdT> ConsensusProtocol for Subset<N, S> {
type NodeId = N;
type Input = Vec<u8>;
type Output = SubsetOutput<N>;

View File

@ -21,7 +21,7 @@ use rand_derive::Rand;
use serde_derive::{Deserialize, Serialize};
use crate::fault_log::{self, Fault};
use crate::{DistAlgorithm, NetworkInfo, NodeIdT, Target};
use crate::{ConsensusProtocol, NetworkInfo, NodeIdT, Target};
/// A threshold decryption error.
#[derive(Clone, Eq, PartialEq, Debug, Fail)]
@ -80,9 +80,9 @@ pub struct ThresholdDecrypt<N> {
}
/// A `ThresholdDecrypt` step. It will contain at most one output.
pub type Step<N> = crate::DaStep<ThresholdDecrypt<N>>;
pub type Step<N> = crate::CpStep<ThresholdDecrypt<N>>;
impl<N: NodeIdT> DistAlgorithm for ThresholdDecrypt<N> {
impl<N: NodeIdT> ConsensusProtocol for ThresholdDecrypt<N> {
type NodeId = N;
type Input = ();
type Output = Vec<u8>;

View File

@ -27,7 +27,7 @@ use rand_derive::Rand;
use serde_derive::{Deserialize, Serialize};
use crate::fault_log::{Fault, FaultLog};
use crate::{DistAlgorithm, NetworkInfo, NodeIdT, Target};
use crate::{ConsensusProtocol, NetworkInfo, NodeIdT, Target};
/// A threshold signing error.
#[derive(Clone, Eq, PartialEq, Debug, Fail)]
@ -89,9 +89,9 @@ pub struct ThresholdSign<N> {
}
/// A step returned from `ThresholdSign`. It contains at most one output.
pub type Step<N> = crate::DaStep<ThresholdSign<N>>;
pub type Step<N> = crate::CpStep<ThresholdSign<N>>;
impl<N: NodeIdT> DistAlgorithm for ThresholdSign<N> {
impl<N: NodeIdT> ConsensusProtocol for ThresholdSign<N> {
type NodeId = N;
type Input = ();
type Output = Signature;

View File

@ -1,4 +1,4 @@
//! Common supertraits for distributed algorithms.
//! Common supertraits for consensus protocols.
use std::collections::BTreeMap;
use std::fmt::{Debug, Display};
@ -21,7 +21,7 @@ impl<C> Contribution for C where C: Eq + Debug + Hash + Send + Sync {}
pub trait NodeIdT: Eq + Ord + Clone + Debug + Hash + Send + Sync {}
impl<N> NodeIdT for N where N: Eq + Ord + Clone + Debug + Hash + Send + Sync {}
/// A distributed algorithm fault.
/// A consensus protocol fault.
pub trait FaultT: Clone + Debug + Fail + PartialEq {}
impl<N> FaultT for N where N: Clone + Debug + Fail + PartialEq {}
@ -230,11 +230,11 @@ pub trait Epoched {
}
/// An alias for the type of `Step` returned by `D`'s methods.
pub type DaStep<D> = Step<
<D as DistAlgorithm>::Message,
<D as DistAlgorithm>::Output,
<D as DistAlgorithm>::NodeId,
<D as DistAlgorithm>::FaultKind,
pub type CpStep<D> = Step<
<D as ConsensusProtocol>::Message,
<D as ConsensusProtocol>::Output,
<D as ConsensusProtocol>::NodeId,
<D as ConsensusProtocol>::FaultKind,
>;
impl<'i, M, O, N, F> Step<M, O, N, F>
@ -290,11 +290,11 @@ where
}
}
/// A distributed algorithm that defines a message flow.
/// A consensus protocol that defines a message flow.
///
/// Many algorithms require an RNG which must be supplied on each call. It is up to the caller to
/// ensure that this random number generator is cryptographically secure.
pub trait DistAlgorithm: Send + Sync {
pub trait ConsensusProtocol: Send + Sync {
/// Unique node identifier.
type NodeId: NodeIdT;
/// The input provided by the user.
@ -314,7 +314,7 @@ pub trait DistAlgorithm: Send + Sync {
&mut self,
input: Self::Input,
rng: &mut R,
) -> Result<DaStep<Self>, Self::Error>
) -> Result<CpStep<Self>, Self::Error>
where
Self: Sized;
@ -324,7 +324,7 @@ pub trait DistAlgorithm: Send + Sync {
sender_id: &Self::NodeId,
message: Self::Message,
rng: &mut R,
) -> Result<DaStep<Self>, Self::Error>
) -> Result<CpStep<Self>, Self::Error>
where
Self: Sized;

View File

@ -10,7 +10,7 @@ The new implementation offers many advantages, such as better abstractions for a
## VirtualNet
Core of most tests is the `net::VirtualNet` struct, which simulates a network of nodes all running an instance of a distributed algorithm. Messages sent by these nodes are queued by the network and dispatched each time the network is advancing one iteration, commonly referred to as being "cranked". Each time the network is cranked, a buffered message is delivered to its destination node and processed.
Core of most tests is the `net::VirtualNet` struct, which simulates a network of nodes all running an instance of a consensus protocol. Messages sent by these nodes are queued by the network and dispatched each time the network is advancing one iteration, commonly referred to as being "cranked". Each time the network is cranked, a buffered message is delivered to its destination node and processed.
Virtual networks can also host an adversary that can affect faulty nodes (which are tracked automatically) or reorder queued messages.

View File

@ -24,7 +24,7 @@ use proptest::{prelude::ProptestConfig, prop_compose, proptest, proptest_helper}
use rand::{Rng, SeedableRng};
use hbbft::binary_agreement::BinaryAgreement;
use hbbft::DistAlgorithm;
use hbbft::ConsensusProtocol;
use crate::net::adversary::{Adversary, ReorderingAdversary};
use crate::net::proptest::{gen_seed, NetworkDimension, TestRng, TestRngSeed};

View File

@ -8,7 +8,7 @@ use std::sync::{Arc, Mutex};
use hbbft::binary_agreement::{BinaryAgreement, MessageContent, SbvMessage};
use hbbft::threshold_sign::ThresholdSign;
use hbbft::{DaStep, DistAlgorithm, NetworkInfo};
use hbbft::{ConsensusProtocol, CpStep, NetworkInfo};
use proptest::{proptest, proptest_helper};
use rand::{Rng, SeedableRng};
@ -417,7 +417,7 @@ impl Adversary<Algo> for AbaCommonCoinAdversary {
_: NetMutHandle<Algo, Self>,
msg: NetMessage<Algo>,
_rng: &mut R,
) -> Result<DaStep<Algo>, CrankError<Algo>> {
) -> Result<CpStep<Algo>, CrankError<Algo>> {
if let MessageContent::Coin(ref coin_msg) = msg.payload().content {
let mut new_coin_state = None;
if let CoinState::InProgress(ref mut coin) = self.coin_state {
@ -432,7 +432,7 @@ impl Adversary<Algo> for AbaCommonCoinAdversary {
self.coin_state = new_coin_state;
}
}
Ok(DaStep::<Algo>::default())
Ok(CpStep::<Algo>::default())
}
}

View File

@ -15,7 +15,7 @@ use crate::network::{
TestNetwork, TestNode,
};
use hbbft::broadcast::{Broadcast, Message};
use hbbft::{util, DistAlgorithm, NetworkInfo, Target, TargetedMessage};
use hbbft::{util, ConsensusProtocol, NetworkInfo, Target, TargetedMessage};
/// An adversary that inputs an alternate value.
struct ProposeAdversary {

View File

@ -14,7 +14,7 @@ use rand::{seq::SliceRandom, Rng};
use hbbft::honey_badger::{Batch, EncryptionSchedule, HoneyBadger, MessageContent};
use hbbft::sender_queue::{self, SenderQueue, Step};
use hbbft::transaction_queue::TransactionQueue;
use hbbft::{threshold_decrypt, util, DistAlgorithm, NetworkInfo, Target, TargetedMessage};
use hbbft::{threshold_decrypt, util, ConsensusProtocol, NetworkInfo, Target, TargetedMessage};
use crate::network::{
Adversary, MessageScheduler, MessageWithSender, NodeId, RandomAdversary, SilentAdversary,
@ -58,7 +58,7 @@ impl Adversary<UsizeHoneyBadger> for FaultyShareAdversary {
fn push_message(
&mut self,
sender_id: NodeId,
msg: TargetedMessage<<UsizeHoneyBadger as DistAlgorithm>::Message, NodeId>,
msg: TargetedMessage<<UsizeHoneyBadger as ConsensusProtocol>::Message, NodeId>,
) {
let NodeId(sender_id) = sender_id;
if sender_id < self.num_good {

View File

@ -4,7 +4,7 @@
//! node. These functions are handled through callbacks, implemented individually by each adversary.
//!
//! This module contains algorithm-agnostic adversaries, which should work for (or rather, against)
//! any `DistAlgorithm`. Specific adversaries tailored to individual algorithms are implemented
//! any `ConsensusProtocol`. Specific adversaries tailored to individual algorithms are implemented
//! alongside their other test cases.
//!
//! ## Adversary model
@ -38,7 +38,7 @@ use std::collections::VecDeque;
use rand::Rng;
use hbbft::{DaStep, DistAlgorithm};
use hbbft::{ConsensusProtocol, CpStep};
use crate::net::{CrankError, NetMessage, Node, VirtualNet};
@ -48,14 +48,14 @@ use crate::net::{CrankError, NetMessage, Node, VirtualNet};
#[derive(Debug)]
pub struct NetHandle<'a, D: 'a, A>(&'a VirtualNet<D, A>)
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>;
impl<'a, D: 'a, A> NetHandle<'a, D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>,
@ -114,14 +114,14 @@ pub enum QueuePosition {
#[derive(Debug)]
pub struct NetMutHandle<'a, D: 'a, A>(&'a mut VirtualNet<D, A>)
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>;
impl<'a, D, A> NetMutHandle<'a, D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
A: Adversary<D>,
D::NodeId: Clone,
D::Message: Clone,
@ -157,7 +157,7 @@ where
&mut self,
msg: NetMessage<D>,
rng: &mut R,
) -> Result<DaStep<D>, CrankError<D>> {
) -> Result<CpStep<D>, CrankError<D>> {
self.0.dispatch_message(msg, rng)
}
@ -227,7 +227,7 @@ where
// Downgrade-conversion.
impl<'a, D, A> From<NetMutHandle<'a, D, A>> for NetHandle<'a, D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
A: Adversary<D>,
D::Message: Clone,
D::Output: Clone,
@ -242,11 +242,11 @@ where
#[derive(Debug)]
pub struct NodeHandle<'a, D: 'a>(&'a Node<D>)
where
D: DistAlgorithm;
D: ConsensusProtocol;
impl<'a, D> NodeHandle<'a, D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
/// Construct a new immutable node handle.
#[inline]
@ -286,11 +286,11 @@ where
#[derive(Debug)]
pub struct NodeMutHandle<'a, D: 'a>(&'a mut Node<D>)
where
D: DistAlgorithm;
D: ConsensusProtocol;
impl<'a, D: 'a> NodeMutHandle<'a, D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
/// Construct a new mutable node handle.
fn new(inner: &'a mut Node<D>) -> Self {
@ -323,7 +323,7 @@ where
pub trait Adversary<D>
where
Self: Sized,
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
{
@ -355,7 +355,7 @@ where
mut net: NetMutHandle<D, Self>,
msg: NetMessage<D>,
rng: &mut R,
) -> Result<DaStep<D>, CrankError<D>> {
) -> Result<CpStep<D>, CrankError<D>> {
net.dispatch_message(msg, rng)
}
}
@ -377,7 +377,7 @@ impl NullAdversary {
impl<D> Adversary<D> for NullAdversary
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
{
@ -402,7 +402,7 @@ impl NodeOrderAdversary {
impl<D> Adversary<D> for NodeOrderAdversary
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
{
@ -429,7 +429,7 @@ impl ReorderingAdversary {
impl<D> Adversary<D> for ReorderingAdversary
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
{

View File

@ -6,7 +6,7 @@ use std::time;
use failure;
use threshold_crypto as crypto;
use hbbft::DistAlgorithm;
use hbbft::ConsensusProtocol;
use super::NetMessage;
@ -15,14 +15,14 @@ use super::NetMessage;
/// Errors resulting from processing a single message ("cranking").
pub enum CrankError<D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
/// The algorithm run by the node produced a `DistAlgorithm::Error` while processing input.
/// The algorithm run by the node produced a `ConsensusProtocol::Error` while processing input.
HandleInput(D::Error),
/// The algorithm run by the node produced a `DistAlgorithm::Error` while processing input to
/// The algorithm run by the node produced a `ConsensusProtocol::Error` while processing input to
/// all nodes.
HandleInputAll(D::Error),
/// The algorithm run by the node produced a `DistAlgorithm::Error` while processing a message.
/// The algorithm run by the node produced a `ConsensusProtocol::Error` while processing a message.
HandleMessage {
/// Network message that triggered the error.
msg: NetMessage<D>,
@ -40,7 +40,7 @@ where
MessageLimitExceeded(usize),
/// The execution time limit has been reached or exceeded.
TimeLimitHit(time::Duration),
/// A `Fault` was reported by a correct node in a step of a `DistAlgorithm`.
/// A `Fault` was reported by a correct node in a step of a `ConsensusProtocol`.
Fault {
/// The ID of the node that reported the fault.
reported_by: D::NodeId,
@ -55,7 +55,7 @@ where
// Note: Deriving [Debug](std::fmt::Debug), [Fail](failure::Fail) and through that,
// [Debug](std::fmt::Debug) automatically does not work due to the wrongly required trait
// bound of `D: DistAlgorithm` implementing the respective Trait. For this reason, these
// bound of `D: ConsensusProtocol` implementing the respective Trait. For this reason, these
// three traits are implemented manually.
//
// More details at
@ -64,7 +64,7 @@ where
// * <https://github.com/rust-lang/rust/issues/26925#issuecomment-405189266>
impl<D> Display for CrankError<D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result {
match self {
@ -120,7 +120,7 @@ where
impl<D> Debug for CrankError<D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result {
match self {
@ -168,7 +168,7 @@ where
impl<D> failure::Fail for CrankError<D>
where
D: DistAlgorithm + 'static,
D: ConsensusProtocol + 'static,
{
fn cause(&self) -> Option<&failure::Fail> {
match self {

View File

@ -27,7 +27,7 @@ use rand::{self, Rng};
use hbbft::dynamic_honey_badger::Batch;
use hbbft::sender_queue::SenderQueueableOutput;
use hbbft::{self, Contribution, DaStep, DistAlgorithm, Fault, NetworkInfo, NodeIdT, Step};
use hbbft::{self, ConsensusProtocol, Contribution, CpStep, Fault, NetworkInfo, NodeIdT, Step};
use crate::try_some;
@ -67,7 +67,7 @@ fn open_trace() -> Result<io::BufWriter<fs::File>, io::Error> {
}
/// A node in the test network.
pub struct Node<D: DistAlgorithm> {
pub struct Node<D: ConsensusProtocol> {
/// Algorithm instance of node.
algorithm: D,
/// Whether or not the node is faulty.
@ -80,7 +80,7 @@ pub struct Node<D: DistAlgorithm> {
impl<D> fmt::Debug for Node<D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result {
f.debug_struct("Node")
@ -91,7 +91,7 @@ where
}
}
impl<D: DistAlgorithm> Node<D> {
impl<D: ConsensusProtocol> Node<D> {
/// Create a new node.
#[inline]
fn new(algorithm: D, is_faulty: bool) -> Self {
@ -146,7 +146,7 @@ impl<D: DistAlgorithm> Node<D> {
}
/// Collects all outputs and faults (not required for network operation) for user convenience.
fn store_step(&mut self, step: &DaStep<D>)
fn store_step(&mut self, step: &CpStep<D>)
where
D::Output: Clone,
{
@ -195,11 +195,11 @@ impl<M, N> NetworkMessage<M, N> {
}
/// Mapping from node IDs to actual node instances.
pub type NodeMap<D> = BTreeMap<<D as DistAlgorithm>::NodeId, Node<D>>;
pub type NodeMap<D> = BTreeMap<<D as ConsensusProtocol>::NodeId, Node<D>>;
/// A virtual network message tied to a distributed algorithm.
/// A virtual network message tied to a consensus protocol.
pub type NetMessage<D> =
NetworkMessage<<D as DistAlgorithm>::Message, <D as DistAlgorithm>::NodeId>;
NetworkMessage<<D as ConsensusProtocol>::Message, <D as ConsensusProtocol>::NodeId>;
/// Process a step.
///
@ -218,12 +218,12 @@ pub type NetMessage<D> =
fn process_step<'a, D>(
nodes: &'a mut BTreeMap<D::NodeId, Node<D>>,
stepped_id: D::NodeId,
step: &DaStep<D>,
step: &CpStep<D>,
dest: &mut VecDeque<NetMessage<D>>,
error_on_fault: bool,
) -> Result<usize, CrankError<D>>
where
D: DistAlgorithm + 'a,
D: ConsensusProtocol + 'a,
D::Message: Clone,
D::Output: Clone,
{
@ -291,7 +291,7 @@ where
#[derive(Debug)]
pub struct NewNodeInfo<D>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
/// The node ID for the new node.
pub id: D::NodeId,
@ -310,14 +310,14 @@ where
/// otherwise the construction will fail and panic.
pub struct NetBuilder<D, I, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
{
/// Iterator used to create node ids.
node_ids: I,
/// Number of faulty nodes in the network.
num_faulty: usize,
/// Dist-algorithm constructor function.
cons: Option<Box<Fn(NewNodeInfo<D>) -> (D, DaStep<D>)>>,
cons: Option<Box<Fn(NewNodeInfo<D>) -> (D, CpStep<D>)>>,
/// Network adversary.
adversary: Option<A>,
/// Trace-enabling flag. `None` means use environment.
@ -335,7 +335,7 @@ where
impl<D, I, A> fmt::Debug for NetBuilder<D, I, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
A: fmt::Debug,
{
fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result {
@ -355,7 +355,7 @@ where
impl<D, I, A> NetBuilder<D, I, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
I: IntoIterator<Item = D::NodeId>,
@ -469,7 +469,7 @@ where
#[inline]
pub fn using_step<F>(mut self, cons: F) -> Self
where
F: Fn(NewNodeInfo<D>) -> (D, DaStep<D>) + 'static,
F: Fn(NewNodeInfo<D>) -> (D, CpStep<D>) + 'static,
{
self.cons = Some(Box::new(cons));
self
@ -498,7 +498,7 @@ where
pub fn build<R: Rng>(
self,
rng: &mut R,
) -> Result<(VirtualNet<D, A>, Vec<(D::NodeId, DaStep<D>)>), CrankError<D>> {
) -> Result<(VirtualNet<D, A>, Vec<(D::NodeId, CpStep<D>)>), CrankError<D>> {
// The time limit can be overriden through environment variables:
let override_time_limit = env::var("HBBFT_NO_TIME_LIMIT")
// We fail early, to avoid tricking the user into thinking that they have set the time
@ -556,14 +556,14 @@ where
///
/// Virtual networks host a number of nodes that are marked either correct or faulty. Each time a
/// node emits a `Step`, the contained messages are queued for delivery, which happens whenever
/// `crank()` is called. Additionally, inputs (see `DistAlgorithm::Input`) can be sent to any node.
/// `crank()` is called. Additionally, inputs (see `ConsensusProtocol::Input`) can be sent to any node.
///
/// An adversary can be hooked into the network to affect the order of message delivery or the
/// behaviour of faulty nodes.
#[derive(Debug)]
pub struct VirtualNet<D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
A: Adversary<D>,
D::Message: Clone,
D::Output: Clone,
@ -599,7 +599,7 @@ where
impl<D, A> VirtualNet<D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>,
@ -724,7 +724,7 @@ where
impl<D, A> VirtualNet<D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>,
@ -733,7 +733,7 @@ where
///
/// Creates a new network from `node_ids`, with the first `faulty` nodes marked faulty. To
/// construct nodes, the `cons` function is passed the ID and the generated `NetworkInfo` and
/// expected to return a (`DistAlgorithm`, `Step`) tuple.
/// expected to return a (`ConsensusProtocol`, `Step`) tuple.
///
/// All messages from the resulting step are queued for delivery. The function outputs the
/// initial steps of the nodes in the constructed network for testing purposes.
@ -750,9 +750,9 @@ where
mut rng: R,
cons: F,
error_on_fault: bool,
) -> Result<(Self, Vec<(D::NodeId, DaStep<D>)>), CrankError<D>>
) -> Result<(Self, Vec<(D::NodeId, CpStep<D>)>), CrankError<D>>
where
F: Fn(NewNodeInfo<D>) -> (D, DaStep<D>),
F: Fn(NewNodeInfo<D>) -> (D, CpStep<D>),
I: IntoIterator<Item = D::NodeId>,
R: rand::Rng,
{
@ -824,7 +824,7 @@ where
&mut self,
msg: NetMessage<D>,
rng: &mut R,
) -> Result<DaStep<D>, CrankError<D>> {
) -> Result<CpStep<D>, CrankError<D>> {
let node = self
.nodes
.get_mut(&msg.to)
@ -856,7 +856,7 @@ where
id: D::NodeId,
input: D::Input,
rng: &mut R,
) -> Result<DaStep<D>, CrankError<D>> {
) -> Result<CpStep<D>, CrankError<D>> {
let step = self
.nodes
.get_mut(&id)
@ -871,7 +871,7 @@ where
/// Processes a step of a given node. The results of the processing are stored internally in the
/// test network.
#[must_use = "The result of processing a step must be used."]
pub fn process_step(&mut self, id: D::NodeId, step: &DaStep<D>) -> Result<(), CrankError<D>> {
pub fn process_step(&mut self, id: D::NodeId, step: &CpStep<D>) -> Result<(), CrankError<D>> {
self.message_count = self.message_count.saturating_add(process_step(
&mut self.nodes,
id,
@ -893,7 +893,7 @@ where
pub fn crank<R: Rng>(
&mut self,
rng: &mut R,
) -> Option<Result<(D::NodeId, DaStep<D>), CrankError<D>>> {
) -> Option<Result<(D::NodeId, CpStep<D>), CrankError<D>>> {
// Check limits.
if let Some(limit) = self.crank_limit {
if self.crank_count >= limit {
@ -984,7 +984,7 @@ where
///
/// Shortcut for cranking the network, expecting both progress to be made as well as processing
/// to proceed.
pub fn crank_expect<R: Rng>(&mut self, rng: &mut R) -> (D::NodeId, DaStep<D>) {
pub fn crank_expect<R: Rng>(&mut self, rng: &mut R) -> (D::NodeId, CpStep<D>) {
self.crank(rng)
.expect("crank: network queue empty")
.expect("crank: node failed to process step")
@ -993,7 +993,7 @@ where
impl<D, A> VirtualNet<D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Input: Clone,
D::Output: Clone,
@ -1010,7 +1010,7 @@ where
&'a mut self,
input: &'a D::Input,
rng: &mut R,
) -> Result<Vec<(D::NodeId, DaStep<D>)>, CrankError<D>> {
) -> Result<Vec<(D::NodeId, CpStep<D>)>, CrankError<D>> {
let steps: Vec<_> = self
.nodes
.values_mut()
@ -1035,7 +1035,7 @@ where
impl<C, D, N, A> VirtualNet<D, A>
where
D: DistAlgorithm<NodeId = N, Output = Batch<C, N>>,
D: ConsensusProtocol<NodeId = N, Output = Batch<C, N>>,
D::Message: Clone,
A: Adversary<D>,
C: Contribution + Clone,
@ -1093,7 +1093,7 @@ where
impl<D, A> ops::Index<D::NodeId> for VirtualNet<D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>,
@ -1108,7 +1108,7 @@ where
impl<D, A> ops::IndexMut<D::NodeId> for VirtualNet<D, A>
where
D: DistAlgorithm,
D: ConsensusProtocol,
D::Message: Clone,
D::Output: Clone,
A: Adversary<D>,

View File

@ -12,7 +12,7 @@ use serde_derive::{Deserialize, Serialize};
use hbbft::dynamic_honey_badger::Batch;
use hbbft::sender_queue::SenderQueueableOutput;
use hbbft::{
Contribution, DaStep, DistAlgorithm, Fault, NetworkInfo, Step, Target, TargetedMessage,
ConsensusProtocol, Contribution, CpStep, Fault, NetworkInfo, Step, Target, TargetedMessage,
};
/// A node identifier. In the tests, nodes are simply numbered.
@ -20,7 +20,7 @@ use hbbft::{
pub struct NodeId(pub usize);
/// A "node" running an instance of the algorithm `D`.
pub struct TestNode<D: DistAlgorithm> {
pub struct TestNode<D: ConsensusProtocol> {
/// This node's own ID.
pub id: D::NodeId,
/// The instance of the broadcast algorithm.
@ -35,7 +35,7 @@ pub struct TestNode<D: DistAlgorithm> {
faults: Vec<Fault<D::NodeId, D::FaultKind>>,
}
impl<D: DistAlgorithm> TestNode<D> {
impl<D: ConsensusProtocol> TestNode<D> {
/// Returns the list of outputs received by this node.
pub fn outputs(&self) -> &[D::Output] {
&self.outputs
@ -68,7 +68,7 @@ impl<D: DistAlgorithm> TestNode<D> {
}
/// Creates a new test node with the given broadcast instance.
fn new((algo, step): (D, DaStep<D>)) -> TestNode<D> {
fn new((algo, step): (D, CpStep<D>)) -> TestNode<D> {
TestNode {
id: algo.our_id().clone(),
algo,
@ -110,7 +110,7 @@ pub enum MessageScheduler {
impl MessageScheduler {
/// Chooses a node to be the next one to handle a message.
pub fn pick_node<D: DistAlgorithm>(
pub fn pick_node<D: ConsensusProtocol>(
&self,
nodes: &BTreeMap<D::NodeId, TestNode<D>>,
) -> D::NodeId {
@ -132,16 +132,16 @@ impl MessageScheduler {
}
/// A message combined with a sender.
pub struct MessageWithSender<D: DistAlgorithm> {
pub struct MessageWithSender<D: ConsensusProtocol> {
/// The sender of the message.
pub sender: <D as DistAlgorithm>::NodeId,
pub sender: <D as ConsensusProtocol>::NodeId,
/// The targeted message (recipient and message body).
pub tm: TargetedMessage<<D as DistAlgorithm>::Message, <D as DistAlgorithm>::NodeId>,
pub tm: TargetedMessage<<D as ConsensusProtocol>::Message, <D as ConsensusProtocol>::NodeId>,
}
// The Debug implementation cannot be derived automatically, possibly due to a compiler bug. For
// this reason, it is implemented manually here.
impl<D: DistAlgorithm> fmt::Debug for MessageWithSender<D> {
impl<D: ConsensusProtocol> fmt::Debug for MessageWithSender<D> {
fn fmt(&self, f: &mut fmt::Formatter) -> fmt::Result {
write!(
f,
@ -151,7 +151,7 @@ impl<D: DistAlgorithm> fmt::Debug for MessageWithSender<D> {
}
}
impl<D: DistAlgorithm> MessageWithSender<D> {
impl<D: ConsensusProtocol> MessageWithSender<D> {
/// Creates a new message with a sender.
pub fn new(
sender: D::NodeId,
@ -164,7 +164,7 @@ impl<D: DistAlgorithm> MessageWithSender<D> {
/// An adversary that can control a set of nodes and pick the next good node to receive a message.
///
/// See `TestNetwork::step()` for a more detailed description of its capabilities.
pub trait Adversary<D: DistAlgorithm> {
pub trait Adversary<D: ConsensusProtocol> {
/// Chooses a node to be the next one to handle a message.
///
/// Starvation is illegal, i.e. in every iteration a node that has pending incoming messages
@ -200,7 +200,7 @@ impl SilentAdversary {
}
}
impl<D: DistAlgorithm> Adversary<D> for SilentAdversary {
impl<D: ConsensusProtocol> Adversary<D> for SilentAdversary {
fn pick_node(&self, nodes: &BTreeMap<D::NodeId, TestNode<D>>) -> D::NodeId {
self.scheduler.pick_node(nodes)
}
@ -234,7 +234,7 @@ fn test_randomly() {
/// The adversary will randomly take a message that is sent to one of its nodes and re-send it to
/// a different node. Additionally, it will inject unrelated messages at random.
#[allow(unused)] // not used in all tests
pub struct RandomAdversary<D: DistAlgorithm, F> {
pub struct RandomAdversary<D: ConsensusProtocol, F> {
/// The underlying scheduler used
scheduler: MessageScheduler,
@ -254,7 +254,7 @@ pub struct RandomAdversary<D: DistAlgorithm, F> {
p_inject: f32,
}
impl<D: DistAlgorithm, F> RandomAdversary<D, F> {
impl<D: ConsensusProtocol, F> RandomAdversary<D, F> {
/// Creates a new random adversary instance.
#[allow(unused)]
pub fn new(p_replay: f32, p_inject: f32, generator: F) -> RandomAdversary<D, F> {
@ -276,7 +276,7 @@ impl<D: DistAlgorithm, F> RandomAdversary<D, F> {
}
}
impl<D: DistAlgorithm, F: Fn() -> TargetedMessage<D::Message, D::NodeId>> Adversary<D>
impl<D: ConsensusProtocol, F: Fn() -> TargetedMessage<D::Message, D::NodeId>> Adversary<D>
for RandomAdversary<D, F>
{
fn init(
@ -361,7 +361,7 @@ impl<D: DistAlgorithm, F: Fn() -> TargetedMessage<D::Message, D::NodeId>> Advers
/// A collection of `TestNode`s representing a network.
///
/// Each `TestNetwork` type is tied to a specific adversary and a distributed algorithm. It consists
/// Each `TestNetwork` type is tied to a specific adversary and a consensus protocol. It consists
/// of a set of nodes, some of which are controlled by the adversary and some of which may be
/// observer nodes, as well as a set of threshold-cryptography public keys.
///
@ -372,14 +372,14 @@ impl<D: DistAlgorithm, F: Fn() -> TargetedMessage<D::Message, D::NodeId>> Advers
/// 2. Send arbitrary messages to any node originating from one of the nodes they control.
///
/// See the `step` function for details on actual operation of the network.
pub struct TestNetwork<A: Adversary<D>, D: DistAlgorithm> {
pub struct TestNetwork<A: Adversary<D>, D: ConsensusProtocol> {
pub nodes: BTreeMap<D::NodeId, TestNode<D>>,
pub observer: TestNode<D>,
pub adv_nodes: BTreeMap<D::NodeId, Arc<NetworkInfo<D::NodeId>>>,
adversary: A,
}
impl<A: Adversary<D>, D: DistAlgorithm<NodeId = NodeId>> TestNetwork<A, D>
impl<A: Adversary<D>, D: ConsensusProtocol<NodeId = NodeId>> TestNetwork<A, D>
where
D::Message: Clone,
{
@ -410,7 +410,7 @@ where
new_algo: F,
) -> TestNetwork<A, D>
where
F: Fn(Arc<NetworkInfo<NodeId>>) -> (D, DaStep<D>),
F: Fn(Arc<NetworkInfo<NodeId>>) -> (D, CpStep<D>),
G: Fn(BTreeMap<D::NodeId, Arc<NetworkInfo<D::NodeId>>>) -> A,
{
let mut rng = rand::thread_rng();
@ -605,7 +605,7 @@ where
impl<A: Adversary<D>, C, D> TestNetwork<A, D>
where
D: DistAlgorithm<Output = Batch<C, NodeId>, NodeId = NodeId>,
D: ConsensusProtocol<Output = Batch<C, NodeId>, NodeId = NodeId>,
C: Contribution + Clone,
{
/// Verifies that all nodes' outputs agree, given a correct "full" node that output all