Remove async-trait from tendermint-machine, tributary-chain

This commit is contained in:
Luke Parker
2025-01-08 16:41:11 -05:00
parent 2121a9b131
commit 376a66b000
9 changed files with 178 additions and 169 deletions

2
Cargo.lock generated
View File

@@ -10498,7 +10498,6 @@ dependencies = [
name = "tendermint-machine" name = "tendermint-machine"
version = "0.2.0" version = "0.2.0"
dependencies = [ dependencies = [
"async-trait",
"futures-channel", "futures-channel",
"futures-util", "futures-util",
"hex", "hex",
@@ -10941,7 +10940,6 @@ dependencies = [
name = "tributary-chain" name = "tributary-chain"
version = "0.1.0" version = "0.1.0"
dependencies = [ dependencies = [
"async-trait",
"blake2", "blake2",
"ciphersuite", "ciphersuite",
"flexible-transcript", "flexible-transcript",

View File

@@ -16,7 +16,6 @@ rustdoc-args = ["--cfg", "docsrs"]
workspace = true workspace = true
[dependencies] [dependencies]
async-trait = { version = "0.1", default-features = false }
thiserror = { version = "2", default-features = false, features = ["std"] } thiserror = { version = "2", default-features = false, features = ["std"] }
subtle = { version = "^2", default-features = false, features = ["std"] } subtle = { version = "^2", default-features = false, features = ["std"] }

View File

@@ -1,8 +1,6 @@
use core::{marker::PhantomData, fmt::Debug}; use core::{marker::PhantomData, fmt::Debug, future::Future};
use std::{sync::Arc, io}; use std::{sync::Arc, io};
use async_trait::async_trait;
use zeroize::Zeroizing; use zeroize::Zeroizing;
use ciphersuite::{Ciphersuite, Ristretto}; use ciphersuite::{Ciphersuite, Ristretto};
@@ -131,20 +129,18 @@ pub trait ReadWrite: Sized {
} }
} }
#[async_trait]
pub trait P2p: 'static + Send + Sync + Clone + Debug { pub trait P2p: 'static + Send + Sync + Clone + Debug {
/// Broadcast a message to all other members of the Tributary with the specified genesis. /// Broadcast a message to all other members of the Tributary with the specified genesis.
/// ///
/// The Tributary will re-broadcast consensus messages on a fixed interval to ensure they aren't /// The Tributary will re-broadcast consensus messages on a fixed interval to ensure they aren't
/// prematurely dropped from the P2P layer. THe P2P layer SHOULD perform content-based /// prematurely dropped from the P2P layer. THe P2P layer SHOULD perform content-based
/// deduplication to ensure a sane amount of load. /// deduplication to ensure a sane amount of load.
async fn broadcast(&self, genesis: [u8; 32], msg: Vec<u8>); fn broadcast(&self, genesis: [u8; 32], msg: Vec<u8>) -> impl Send + Future<Output = ()>;
} }
#[async_trait]
impl<P: P2p> P2p for Arc<P> { impl<P: P2p> P2p for Arc<P> {
async fn broadcast(&self, genesis: [u8; 32], msg: Vec<u8>) { fn broadcast(&self, genesis: [u8; 32], msg: Vec<u8>) -> impl Send + Future<Output = ()> {
(*self).broadcast(genesis, msg).await P::broadcast(self, genesis, msg)
} }
} }

View File

@@ -1,8 +1,6 @@
use core::ops::Deref; use core::{ops::Deref, future::Future};
use std::{sync::Arc, collections::HashMap}; use std::{sync::Arc, collections::HashMap};
use async_trait::async_trait;
use subtle::ConstantTimeEq; use subtle::ConstantTimeEq;
use zeroize::{Zeroize, Zeroizing}; use zeroize::{Zeroize, Zeroizing};
@@ -74,50 +72,52 @@ impl Signer {
} }
} }
#[async_trait]
impl SignerTrait for Signer { impl SignerTrait for Signer {
type ValidatorId = [u8; 32]; type ValidatorId = [u8; 32];
type Signature = [u8; 64]; type Signature = [u8; 64];
/// Returns the validator's current ID. Returns None if they aren't a current validator. /// Returns the validator's current ID. Returns None if they aren't a current validator.
async fn validator_id(&self) -> Option<Self::ValidatorId> { fn validator_id(&self) -> impl Send + Future<Output = Option<Self::ValidatorId>> {
Some((Ristretto::generator() * self.key.deref()).to_bytes()) async move { Some((Ristretto::generator() * self.key.deref()).to_bytes()) }
} }
/// Sign a signature with the current validator's private key. /// Sign a signature with the current validator's private key.
async fn sign(&self, msg: &[u8]) -> Self::Signature { fn sign(&self, msg: &[u8]) -> impl Send + Future<Output = Self::Signature> {
let mut nonce = Zeroizing::new(RecommendedTranscript::new(b"Tributary Chain Tendermint Nonce")); async move {
nonce.append_message(b"genesis", self.genesis); let mut nonce =
nonce.append_message(b"key", Zeroizing::new(self.key.deref().to_repr()).as_ref()); Zeroizing::new(RecommendedTranscript::new(b"Tributary Chain Tendermint Nonce"));
nonce.append_message(b"message", msg); nonce.append_message(b"genesis", self.genesis);
let mut nonce = nonce.challenge(b"nonce"); nonce.append_message(b"key", Zeroizing::new(self.key.deref().to_repr()).as_ref());
nonce.append_message(b"message", msg);
let mut nonce = nonce.challenge(b"nonce");
let mut nonce_arr = [0; 64]; let mut nonce_arr = [0; 64];
nonce_arr.copy_from_slice(nonce.as_ref()); nonce_arr.copy_from_slice(nonce.as_ref());
let nonce_ref: &mut [u8] = nonce.as_mut(); let nonce_ref: &mut [u8] = nonce.as_mut();
nonce_ref.zeroize(); nonce_ref.zeroize();
let nonce_ref: &[u8] = nonce.as_ref(); let nonce_ref: &[u8] = nonce.as_ref();
assert_eq!(nonce_ref, [0; 64].as_ref()); assert_eq!(nonce_ref, [0; 64].as_ref());
let nonce = let nonce =
Zeroizing::new(<Ristretto as Ciphersuite>::F::from_bytes_mod_order_wide(&nonce_arr)); Zeroizing::new(<Ristretto as Ciphersuite>::F::from_bytes_mod_order_wide(&nonce_arr));
nonce_arr.zeroize(); nonce_arr.zeroize();
assert!(!bool::from(nonce.ct_eq(&<Ristretto as Ciphersuite>::F::ZERO))); assert!(!bool::from(nonce.ct_eq(&<Ristretto as Ciphersuite>::F::ZERO)));
let challenge = challenge( let challenge = challenge(
self.genesis, self.genesis,
(Ristretto::generator() * self.key.deref()).to_bytes(), (Ristretto::generator() * self.key.deref()).to_bytes(),
(Ristretto::generator() * nonce.deref()).to_bytes().as_ref(), (Ristretto::generator() * nonce.deref()).to_bytes().as_ref(),
msg, msg,
); );
let sig = SchnorrSignature::<Ristretto>::sign(&self.key, nonce, challenge).serialize(); let sig = SchnorrSignature::<Ristretto>::sign(&self.key, nonce, challenge).serialize();
let mut res = [0; 64]; let mut res = [0; 64];
res.copy_from_slice(&sig); res.copy_from_slice(&sig);
res res
}
} }
} }
@@ -274,7 +274,6 @@ pub const BLOCK_PROCESSING_TIME: u32 = 999;
pub const LATENCY_TIME: u32 = 1667; pub const LATENCY_TIME: u32 = 1667;
pub const TARGET_BLOCK_TIME: u32 = BLOCK_PROCESSING_TIME + (3 * LATENCY_TIME); pub const TARGET_BLOCK_TIME: u32 = BLOCK_PROCESSING_TIME + (3 * LATENCY_TIME);
#[async_trait]
impl<D: Db, T: TransactionTrait, P: P2p> Network for TendermintNetwork<D, T, P> { impl<D: Db, T: TransactionTrait, P: P2p> Network for TendermintNetwork<D, T, P> {
type Db = D; type Db = D;
@@ -300,111 +299,126 @@ impl<D: Db, T: TransactionTrait, P: P2p> Network for TendermintNetwork<D, T, P>
self.validators.clone() self.validators.clone()
} }
async fn broadcast(&mut self, msg: SignedMessageFor<Self>) { fn broadcast(&mut self, msg: SignedMessageFor<Self>) -> impl Send + Future<Output = ()> {
let mut to_broadcast = vec![TENDERMINT_MESSAGE]; async move {
to_broadcast.extend(msg.encode()); let mut to_broadcast = vec![TENDERMINT_MESSAGE];
self.p2p.broadcast(self.genesis, to_broadcast).await to_broadcast.extend(msg.encode());
} self.p2p.broadcast(self.genesis, to_broadcast).await
async fn slash(&mut self, validator: Self::ValidatorId, slash_event: SlashEvent) {
log::error!(
"validator {} triggered a slash event on tributary {} (with evidence: {})",
hex::encode(validator),
hex::encode(self.genesis),
matches!(slash_event, SlashEvent::WithEvidence(_)),
);
let signer = self.signer();
let Some(tx) = (match slash_event {
SlashEvent::WithEvidence(evidence) => {
// create an unsigned evidence tx
Some(TendermintTx::SlashEvidence(evidence))
}
SlashEvent::Id(_reason, _block, _round) => {
// TODO: Increase locally observed slash points
None
}
}) else {
return;
};
// add tx to blockchain and broadcast to peers
let mut to_broadcast = vec![TRANSACTION_MESSAGE];
tx.write(&mut to_broadcast).unwrap();
if self.blockchain.write().await.add_transaction::<Self>(
true,
Transaction::Tendermint(tx),
&self.signature_scheme(),
) == Ok(true)
{
self.p2p.broadcast(signer.genesis, to_broadcast).await;
} }
} }
async fn validate(&self, block: &Self::Block) -> Result<(), TendermintBlockError> { fn slash(
let block = &mut self,
Block::read::<&[u8]>(&mut block.0.as_ref()).map_err(|_| TendermintBlockError::Fatal)?; validator: Self::ValidatorId,
self slash_event: SlashEvent,
.blockchain ) -> impl Send + Future<Output = ()> {
.read() async move {
.await log::error!(
.verify_block::<Self>(&block, &self.signature_scheme(), false) "validator {} triggered a slash event on tributary {} (with evidence: {})",
.map_err(|e| match e { hex::encode(validator),
BlockError::NonLocalProvided(_) => TendermintBlockError::Temporal, hex::encode(self.genesis),
_ => { matches!(slash_event, SlashEvent::WithEvidence(_)),
log::warn!("Tributary Tendermint validate returning BlockError::Fatal due to {e:?}"); );
TendermintBlockError::Fatal
let signer = self.signer();
let Some(tx) = (match slash_event {
SlashEvent::WithEvidence(evidence) => {
// create an unsigned evidence tx
Some(TendermintTx::SlashEvidence(evidence))
} }
}) SlashEvent::Id(_reason, _block, _round) => {
// TODO: Increase locally observed slash points
None
}
}) else {
return;
};
// add tx to blockchain and broadcast to peers
let mut to_broadcast = vec![TRANSACTION_MESSAGE];
tx.write(&mut to_broadcast).unwrap();
if self.blockchain.write().await.add_transaction::<Self>(
true,
Transaction::Tendermint(tx),
&self.signature_scheme(),
) == Ok(true)
{
self.p2p.broadcast(signer.genesis, to_broadcast).await;
}
}
} }
async fn add_block( fn validate(
&self,
block: &Self::Block,
) -> impl Send + Future<Output = Result<(), TendermintBlockError>> {
async move {
let block =
Block::read::<&[u8]>(&mut block.0.as_ref()).map_err(|_| TendermintBlockError::Fatal)?;
self
.blockchain
.read()
.await
.verify_block::<Self>(&block, &self.signature_scheme(), false)
.map_err(|e| match e {
BlockError::NonLocalProvided(_) => TendermintBlockError::Temporal,
_ => {
log::warn!("Tributary Tendermint validate returning BlockError::Fatal due to {e:?}");
TendermintBlockError::Fatal
}
})
}
}
fn add_block(
&mut self, &mut self,
serialized_block: Self::Block, serialized_block: Self::Block,
commit: Commit<Self::SignatureScheme>, commit: Commit<Self::SignatureScheme>,
) -> Option<Self::Block> { ) -> impl Send + Future<Output = Option<Self::Block>> {
let invalid_block = || { async move {
// There's a fatal flaw in the code, it's behind a hard fork, or the validators turned let invalid_block = || {
// malicious // There's a fatal flaw in the code, it's behind a hard fork, or the validators turned
// All justify a halt to then achieve social consensus from // malicious
// TODO: Under multiple validator sets, a small validator set turning malicious knocks // All justify a halt to then achieve social consensus from
// off the entire network. That's an unacceptable DoS. // TODO: Under multiple validator sets, a small validator set turning malicious knocks
panic!("validators added invalid block to tributary {}", hex::encode(self.genesis)); // off the entire network. That's an unacceptable DoS.
}; panic!("validators added invalid block to tributary {}", hex::encode(self.genesis));
};
// Tendermint should only produce valid commits // Tendermint should only produce valid commits
assert!(self.verify_commit(serialized_block.id(), &commit)); assert!(self.verify_commit(serialized_block.id(), &commit));
let Ok(block) = Block::read::<&[u8]>(&mut serialized_block.0.as_ref()) else { let Ok(block) = Block::read::<&[u8]>(&mut serialized_block.0.as_ref()) else {
return invalid_block(); return invalid_block();
}; };
let encoded_commit = commit.encode(); let encoded_commit = commit.encode();
loop { loop {
let block_res = self.blockchain.write().await.add_block::<Self>( let block_res = self.blockchain.write().await.add_block::<Self>(
&block, &block,
encoded_commit.clone(), encoded_commit.clone(),
&self.signature_scheme(), &self.signature_scheme(),
); );
match block_res { match block_res {
Ok(()) => { Ok(()) => {
// If we successfully added this block, break // If we successfully added this block, break
break; break;
}
Err(BlockError::NonLocalProvided(hash)) => {
log::error!(
"missing provided transaction {} which other validators on tributary {} had",
hex::encode(hash),
hex::encode(self.genesis)
);
tokio::time::sleep(core::time::Duration::from_secs(5)).await;
}
_ => return invalid_block(),
} }
Err(BlockError::NonLocalProvided(hash)) => {
log::error!(
"missing provided transaction {} which other validators on tributary {} had",
hex::encode(hash),
hex::encode(self.genesis)
);
tokio::time::sleep(core::time::Duration::from_secs(5)).await;
}
_ => return invalid_block(),
} }
}
Some(TendermintBlock( Some(TendermintBlock(
self.blockchain.write().await.build_block::<Self>(&self.signature_scheme()).serialize(), self.blockchain.write().await.build_block::<Self>(&self.signature_scheme()).serialize(),
)) ))
}
} }
} }

View File

@@ -1,11 +1,12 @@
use core::future::Future;
pub use crate::P2p; pub use crate::P2p;
#[derive(Clone, Debug)] #[derive(Clone, Debug)]
pub struct DummyP2p; pub struct DummyP2p;
#[async_trait::async_trait]
impl P2p for DummyP2p { impl P2p for DummyP2p {
async fn broadcast(&self, _: [u8; 32], _: Vec<u8>) { fn broadcast(&self, _: [u8; 32], _: Vec<u8>) -> impl Send + Future<Output = ()> {
unimplemented!() async move { unimplemented!() }
} }
} }

View File

@@ -1,4 +1,7 @@
use core::future::Future;
use tendermint::ext::Network; use tendermint::ext::Network;
use crate::{ use crate::{
P2p, TendermintTx, P2p, TendermintTx,
tendermint::{TARGET_BLOCK_TIME, TendermintNetwork}, tendermint::{TARGET_BLOCK_TIME, TendermintNetwork},
@@ -11,10 +14,9 @@ fn assert_target_block_time() {
#[derive(Clone, Debug)] #[derive(Clone, Debug)]
pub struct DummyP2p; pub struct DummyP2p;
#[async_trait::async_trait]
impl P2p for DummyP2p { impl P2p for DummyP2p {
async fn broadcast(&self, _: [u8; 32], _: Vec<u8>) { fn broadcast(&self, _: [u8; 32], _: Vec<u8>) -> impl Send + Future<Output = ()> {
unimplemented!() async move { unimplemented!() }
} }
} }

View File

@@ -16,7 +16,6 @@ rustdoc-args = ["--cfg", "docsrs"]
workspace = true workspace = true
[dependencies] [dependencies]
async-trait = { version = "0.1", default-features = false }
thiserror = { version = "2", default-features = false, features = ["std"] } thiserror = { version = "2", default-features = false, features = ["std"] }
hex = { version = "0.4", default-features = false, features = ["std"] } hex = { version = "0.4", default-features = false, features = ["std"] }

View File

@@ -1,7 +1,6 @@
use core::{hash::Hash, fmt::Debug}; use core::{hash::Hash, fmt::Debug, future::Future};
use std::{sync::Arc, collections::HashSet}; use std::{sync::Arc, collections::HashSet};
use async_trait::async_trait;
use thiserror::Error; use thiserror::Error;
use parity_scale_codec::{Encode, Decode}; use parity_scale_codec::{Encode, Decode};
@@ -34,7 +33,6 @@ pub struct BlockNumber(pub u64);
pub struct RoundNumber(pub u32); pub struct RoundNumber(pub u32);
/// A signer for a validator. /// A signer for a validator.
#[async_trait]
pub trait Signer: Send + Sync { pub trait Signer: Send + Sync {
// Type used to identify validators. // Type used to identify validators.
type ValidatorId: ValidatorId; type ValidatorId: ValidatorId;
@@ -42,22 +40,21 @@ pub trait Signer: Send + Sync {
type Signature: Signature; type Signature: Signature;
/// Returns the validator's current ID. Returns None if they aren't a current validator. /// Returns the validator's current ID. Returns None if they aren't a current validator.
async fn validator_id(&self) -> Option<Self::ValidatorId>; fn validator_id(&self) -> impl Send + Future<Output = Option<Self::ValidatorId>>;
/// Sign a signature with the current validator's private key. /// Sign a signature with the current validator's private key.
async fn sign(&self, msg: &[u8]) -> Self::Signature; fn sign(&self, msg: &[u8]) -> impl Send + Future<Output = Self::Signature>;
} }
#[async_trait]
impl<S: Signer> Signer for Arc<S> { impl<S: Signer> Signer for Arc<S> {
type ValidatorId = S::ValidatorId; type ValidatorId = S::ValidatorId;
type Signature = S::Signature; type Signature = S::Signature;
async fn validator_id(&self) -> Option<Self::ValidatorId> { fn validator_id(&self) -> impl Send + Future<Output = Option<Self::ValidatorId>> {
self.as_ref().validator_id().await self.as_ref().validator_id()
} }
async fn sign(&self, msg: &[u8]) -> Self::Signature { fn sign(&self, msg: &[u8]) -> impl Send + Future<Output = Self::Signature> {
self.as_ref().sign(msg).await self.as_ref().sign(msg)
} }
} }
@@ -210,7 +207,6 @@ pub trait Block: Send + Sync + Clone + PartialEq + Eq + Debug + Encode + Decode
} }
/// Trait representing the distributed system Tendermint is providing consensus over. /// Trait representing the distributed system Tendermint is providing consensus over.
#[async_trait]
pub trait Network: Sized + Send + Sync { pub trait Network: Sized + Send + Sync {
/// The database used to back this. /// The database used to back this.
type Db: serai_db::Db; type Db: serai_db::Db;
@@ -229,6 +225,7 @@ pub trait Network: Sized + Send + Sync {
/// This should include both the time to download the block and the actual processing time. /// This should include both the time to download the block and the actual processing time.
/// ///
/// BLOCK_PROCESSING_TIME + (3 * LATENCY_TIME) must be divisible by 1000. /// BLOCK_PROCESSING_TIME + (3 * LATENCY_TIME) must be divisible by 1000.
// TODO: Redefine as Duration
const BLOCK_PROCESSING_TIME: u32; const BLOCK_PROCESSING_TIME: u32;
/// Network latency time in milliseconds. /// Network latency time in milliseconds.
/// ///
@@ -280,15 +277,19 @@ pub trait Network: Sized + Send + Sync {
/// Switching to unauthenticated channels in a system already providing authenticated channels is /// Switching to unauthenticated channels in a system already providing authenticated channels is
/// not recommended as this is a minor, temporal inefficiency, while downgrading channels may /// not recommended as this is a minor, temporal inefficiency, while downgrading channels may
/// have wider implications. /// have wider implications.
async fn broadcast(&mut self, msg: SignedMessageFor<Self>); fn broadcast(&mut self, msg: SignedMessageFor<Self>) -> impl Send + Future<Output = ()>;
/// Trigger a slash for the validator in question who was definitively malicious. /// Trigger a slash for the validator in question who was definitively malicious.
/// ///
/// The exact process of triggering a slash is undefined and left to the network as a whole. /// The exact process of triggering a slash is undefined and left to the network as a whole.
async fn slash(&mut self, validator: Self::ValidatorId, slash_event: SlashEvent); fn slash(
&mut self,
validator: Self::ValidatorId,
slash_event: SlashEvent,
) -> impl Send + Future<Output = ()>;
/// Validate a block. /// Validate a block.
async fn validate(&self, block: &Self::Block) -> Result<(), BlockError>; fn validate(&self, block: &Self::Block) -> impl Send + Future<Output = Result<(), BlockError>>;
/// Add a block, returning the proposal for the next one. /// Add a block, returning the proposal for the next one.
/// ///
@@ -298,9 +299,9 @@ pub trait Network: Sized + Send + Sync {
/// This deviates from the paper which will have a local node refuse to decide on a block it /// This deviates from the paper which will have a local node refuse to decide on a block it
/// considers invalid. This library acknowledges the network did decide on it, leaving handling /// considers invalid. This library acknowledges the network did decide on it, leaving handling
/// of it to the network, and outside of this scope. /// of it to the network, and outside of this scope.
async fn add_block( fn add_block(
&mut self, &mut self,
block: Self::Block, block: Self::Block,
commit: Commit<Self::SignatureScheme>, commit: Commit<Self::SignatureScheme>,
) -> Option<Self::Block>; ) -> impl Send + Future<Output = Option<Self::Block>>;
} }

View File

@@ -1,10 +1,9 @@
use core::future::Future;
use std::{ use std::{
sync::Arc, sync::Arc,
time::{UNIX_EPOCH, SystemTime, Duration}, time::{UNIX_EPOCH, SystemTime, Duration},
}; };
use async_trait::async_trait;
use parity_scale_codec::{Encode, Decode}; use parity_scale_codec::{Encode, Decode};
use futures_util::sink::SinkExt; use futures_util::sink::SinkExt;
@@ -21,20 +20,21 @@ type TestValidatorId = u16;
type TestBlockId = [u8; 4]; type TestBlockId = [u8; 4];
struct TestSigner(u16); struct TestSigner(u16);
#[async_trait]
impl Signer for TestSigner { impl Signer for TestSigner {
type ValidatorId = TestValidatorId; type ValidatorId = TestValidatorId;
type Signature = [u8; 32]; type Signature = [u8; 32];
async fn validator_id(&self) -> Option<TestValidatorId> { fn validator_id(&self) -> impl Send + Future<Output = Option<TestValidatorId>> {
Some(self.0) async move { Some(self.0) }
} }
async fn sign(&self, msg: &[u8]) -> [u8; 32] { fn sign(&self, msg: &[u8]) -> impl Send + Future<Output = [u8; 32]> {
let mut sig = [0; 32]; async move {
sig[.. 2].copy_from_slice(&self.0.to_le_bytes()); let mut sig = [0; 32];
sig[2 .. (2 + 30.min(msg.len()))].copy_from_slice(&msg[.. 30.min(msg.len())]); sig[.. 2].copy_from_slice(&self.0.to_le_bytes());
sig sig[2 .. (2 + 30.min(msg.len()))].copy_from_slice(&msg[.. 30.min(msg.len())]);
sig
}
} }
} }
@@ -111,7 +111,6 @@ struct TestNetwork(
Arc<RwLock<Vec<(MessageSender<Self>, SyncedBlockSender<Self>, SyncedBlockResultReceiver)>>>, Arc<RwLock<Vec<(MessageSender<Self>, SyncedBlockSender<Self>, SyncedBlockResultReceiver)>>>,
); );
#[async_trait]
impl Network for TestNetwork { impl Network for TestNetwork {
type Db = MemDb; type Db = MemDb;