mirror of
https://github.com/hl-archive-node/nanoreth.git
synced 2025-12-06 10:59:55 +00:00
refactor: engine interceptors (#8048)
This commit is contained in:
3
Cargo.lock
generated
3
Cargo.lock
generated
@ -7196,6 +7196,7 @@ dependencies = [
|
|||||||
"reth-transaction-pool",
|
"reth-transaction-pool",
|
||||||
"tempfile",
|
"tempfile",
|
||||||
"tokio",
|
"tokio",
|
||||||
|
"tokio-stream",
|
||||||
]
|
]
|
||||||
|
|
||||||
[[package]]
|
[[package]]
|
||||||
@ -7218,6 +7219,7 @@ dependencies = [
|
|||||||
"metrics-process",
|
"metrics-process",
|
||||||
"metrics-util",
|
"metrics-util",
|
||||||
"once_cell",
|
"once_cell",
|
||||||
|
"pin-project",
|
||||||
"procfs",
|
"procfs",
|
||||||
"proptest",
|
"proptest",
|
||||||
"rand 0.8.5",
|
"rand 0.8.5",
|
||||||
@ -7256,6 +7258,7 @@ dependencies = [
|
|||||||
"thiserror",
|
"thiserror",
|
||||||
"tikv-jemalloc-ctl",
|
"tikv-jemalloc-ctl",
|
||||||
"tokio",
|
"tokio",
|
||||||
|
"tokio-util",
|
||||||
"tracing",
|
"tracing",
|
||||||
"vergen",
|
"vergen",
|
||||||
]
|
]
|
||||||
|
|||||||
@ -19,7 +19,7 @@ use reth_consensus::Consensus;
|
|||||||
use reth_db::{init_db, DatabaseEnv};
|
use reth_db::{init_db, DatabaseEnv};
|
||||||
use reth_network::NetworkHandle;
|
use reth_network::NetworkHandle;
|
||||||
use reth_network_api::NetworkInfo;
|
use reth_network_api::NetworkInfo;
|
||||||
use reth_node_core::engine_api_store::{EngineApiStore, StoredEngineApiMessage};
|
use reth_node_core::engine::engine_store::{EngineMessageStore, StoredEngineApiMessage};
|
||||||
#[cfg(not(feature = "optimism"))]
|
#[cfg(not(feature = "optimism"))]
|
||||||
use reth_node_ethereum::{EthEngineTypes, EthEvmConfig};
|
use reth_node_ethereum::{EthEngineTypes, EthEvmConfig};
|
||||||
use reth_payload_builder::{PayloadBuilderHandle, PayloadBuilderService};
|
use reth_payload_builder::{PayloadBuilderHandle, PayloadBuilderService};
|
||||||
@ -34,7 +34,7 @@ use reth_static_file::StaticFileProducer;
|
|||||||
use reth_tasks::TaskExecutor;
|
use reth_tasks::TaskExecutor;
|
||||||
use reth_transaction_pool::noop::NoopTransactionPool;
|
use reth_transaction_pool::noop::NoopTransactionPool;
|
||||||
use std::{net::SocketAddr, path::PathBuf, sync::Arc, time::Duration};
|
use std::{net::SocketAddr, path::PathBuf, sync::Arc, time::Duration};
|
||||||
use tokio::sync::{mpsc, oneshot};
|
use tokio::sync::oneshot;
|
||||||
use tracing::*;
|
use tracing::*;
|
||||||
|
|
||||||
/// `reth debug replay-engine` command
|
/// `reth debug replay-engine` command
|
||||||
@ -191,8 +191,7 @@ impl Command {
|
|||||||
|
|
||||||
// Configure the consensus engine
|
// Configure the consensus engine
|
||||||
let network_client = network.fetch_client().await?;
|
let network_client = network.fetch_client().await?;
|
||||||
let (consensus_engine_tx, consensus_engine_rx) = mpsc::unbounded_channel();
|
let (beacon_consensus_engine, beacon_engine_handle) = BeaconConsensusEngine::new(
|
||||||
let (beacon_consensus_engine, beacon_engine_handle) = BeaconConsensusEngine::with_channel(
|
|
||||||
network_client,
|
network_client,
|
||||||
Pipeline::builder().build(
|
Pipeline::builder().build(
|
||||||
provider_factory.clone(),
|
provider_factory.clone(),
|
||||||
@ -210,8 +209,6 @@ impl Command {
|
|||||||
payload_builder,
|
payload_builder,
|
||||||
None,
|
None,
|
||||||
u64::MAX,
|
u64::MAX,
|
||||||
consensus_engine_tx,
|
|
||||||
consensus_engine_rx,
|
|
||||||
EngineHooks::new(),
|
EngineHooks::new(),
|
||||||
)?;
|
)?;
|
||||||
info!(target: "reth::cli", "Consensus engine initialized");
|
info!(target: "reth::cli", "Consensus engine initialized");
|
||||||
@ -224,7 +221,7 @@ impl Command {
|
|||||||
let _ = tx.send(res);
|
let _ = tx.send(res);
|
||||||
});
|
});
|
||||||
|
|
||||||
let engine_api_store = EngineApiStore::new(self.engine_api_store.clone());
|
let engine_api_store = EngineMessageStore::new(self.engine_api_store.clone());
|
||||||
for filepath in engine_api_store.engine_messages_iter()? {
|
for filepath in engine_api_store.engine_messages_iter()? {
|
||||||
let contents =
|
let contents =
|
||||||
fs::read(&filepath).wrap_err(format!("failed to read: {}", filepath.display()))?;
|
fs::read(&filepath).wrap_err(format!("failed to read: {}", filepath.display()))?;
|
||||||
|
|||||||
@ -1,18 +1,10 @@
|
|||||||
use crate::{
|
use futures::{stream::BoxStream, Future, StreamExt};
|
||||||
engine::{
|
|
||||||
forkchoice::{ForkchoiceStateHash, ForkchoiceStateTracker},
|
|
||||||
metrics::EngineMetrics,
|
|
||||||
},
|
|
||||||
hooks::{EngineHookContext, EngineHooksController},
|
|
||||||
sync::{EngineSyncController, EngineSyncEvent},
|
|
||||||
};
|
|
||||||
use futures::{Future, StreamExt};
|
|
||||||
use reth_db::database::Database;
|
use reth_db::database::Database;
|
||||||
use reth_engine_primitives::{EngineTypes, PayloadAttributes, PayloadBuilderAttributes};
|
use reth_engine_primitives::{EngineTypes, PayloadAttributes, PayloadBuilderAttributes};
|
||||||
use reth_interfaces::{
|
use reth_interfaces::{
|
||||||
blockchain_tree::{
|
blockchain_tree::{
|
||||||
error::{BlockchainTreeError, CanonicalError, InsertBlockError, InsertBlockErrorKind},
|
error::{BlockchainTreeError, CanonicalError, InsertBlockError, InsertBlockErrorKind},
|
||||||
BlockStatus, BlockchainTreeEngine, CanonicalOutcome, InsertPayloadOk,
|
BlockStatus, BlockValidationKind, BlockchainTreeEngine, CanonicalOutcome, InsertPayloadOk,
|
||||||
},
|
},
|
||||||
executor::BlockValidationError,
|
executor::BlockValidationError,
|
||||||
p2p::{bodies::client::BodiesClient, headers::client::HeadersClient},
|
p2p::{bodies::client::BodiesClient, headers::client::HeadersClient},
|
||||||
@ -21,6 +13,7 @@ use reth_interfaces::{
|
|||||||
RethError, RethResult,
|
RethError, RethResult,
|
||||||
};
|
};
|
||||||
use reth_payload_builder::PayloadBuilderHandle;
|
use reth_payload_builder::PayloadBuilderHandle;
|
||||||
|
use reth_payload_validator::ExecutionPayloadValidator;
|
||||||
use reth_primitives::{
|
use reth_primitives::{
|
||||||
constants::EPOCH_SLOTS, stage::StageId, BlockNumHash, BlockNumber, Head, Header, SealedBlock,
|
constants::EPOCH_SLOTS, stage::StageId, BlockNumHash, BlockNumber, Head, Header, SealedBlock,
|
||||||
SealedHeader, B256,
|
SealedHeader, B256,
|
||||||
@ -43,7 +36,7 @@ use std::{
|
|||||||
time::{Duration, Instant},
|
time::{Duration, Instant},
|
||||||
};
|
};
|
||||||
use tokio::sync::{
|
use tokio::sync::{
|
||||||
mpsc::{self, UnboundedReceiver, UnboundedSender},
|
mpsc::{self, UnboundedSender},
|
||||||
oneshot,
|
oneshot,
|
||||||
};
|
};
|
||||||
use tokio_stream::wrappers::UnboundedReceiverStream;
|
use tokio_stream::wrappers::UnboundedReceiverStream;
|
||||||
@ -68,18 +61,19 @@ mod handle;
|
|||||||
pub use handle::BeaconConsensusEngineHandle;
|
pub use handle::BeaconConsensusEngineHandle;
|
||||||
|
|
||||||
mod forkchoice;
|
mod forkchoice;
|
||||||
use crate::hooks::{EngineHookEvent, EngineHooks, PolledHook};
|
|
||||||
pub use forkchoice::ForkchoiceStatus;
|
pub use forkchoice::ForkchoiceStatus;
|
||||||
use reth_interfaces::blockchain_tree::BlockValidationKind;
|
use forkchoice::{ForkchoiceStateHash, ForkchoiceStateTracker};
|
||||||
use reth_payload_validator::ExecutionPayloadValidator;
|
|
||||||
|
|
||||||
mod metrics;
|
mod metrics;
|
||||||
|
use metrics::EngineMetrics;
|
||||||
|
|
||||||
pub(crate) mod sync;
|
pub(crate) mod sync;
|
||||||
|
use sync::{EngineSyncController, EngineSyncEvent};
|
||||||
|
|
||||||
/// Hooks for running during the main loop of
|
/// Hooks for running during the main loop of
|
||||||
/// [consensus engine][`crate::engine::BeaconConsensusEngine`].
|
/// [consensus engine][`crate::engine::BeaconConsensusEngine`].
|
||||||
pub mod hooks;
|
pub mod hooks;
|
||||||
|
use hooks::{EngineHookContext, EngineHookEvent, EngineHooks, EngineHooksController, PolledHook};
|
||||||
|
|
||||||
#[cfg(test)]
|
#[cfg(test)]
|
||||||
pub mod test_utils;
|
pub mod test_utils;
|
||||||
@ -180,7 +174,7 @@ where
|
|||||||
/// Used for emitting updates about whether the engine is syncing or not.
|
/// Used for emitting updates about whether the engine is syncing or not.
|
||||||
sync_state_updater: Box<dyn NetworkSyncUpdater>,
|
sync_state_updater: Box<dyn NetworkSyncUpdater>,
|
||||||
/// The Engine API message receiver.
|
/// The Engine API message receiver.
|
||||||
engine_message_rx: UnboundedReceiverStream<BeaconEngineMessage<EngineT>>,
|
engine_message_stream: BoxStream<'static, BeaconEngineMessage<EngineT>>,
|
||||||
/// A clone of the handle
|
/// A clone of the handle
|
||||||
handle: BeaconConsensusEngineHandle<EngineT>,
|
handle: BeaconConsensusEngineHandle<EngineT>,
|
||||||
/// Tracks the received forkchoice state updates received by the CL.
|
/// Tracks the received forkchoice state updates received by the CL.
|
||||||
@ -254,7 +248,7 @@ where
|
|||||||
target,
|
target,
|
||||||
pipeline_run_threshold,
|
pipeline_run_threshold,
|
||||||
to_engine,
|
to_engine,
|
||||||
rx,
|
Box::pin(UnboundedReceiverStream::from(rx)),
|
||||||
hooks,
|
hooks,
|
||||||
)
|
)
|
||||||
}
|
}
|
||||||
@ -284,7 +278,7 @@ where
|
|||||||
target: Option<B256>,
|
target: Option<B256>,
|
||||||
pipeline_run_threshold: u64,
|
pipeline_run_threshold: u64,
|
||||||
to_engine: UnboundedSender<BeaconEngineMessage<EngineT>>,
|
to_engine: UnboundedSender<BeaconEngineMessage<EngineT>>,
|
||||||
rx: UnboundedReceiver<BeaconEngineMessage<EngineT>>,
|
engine_message_stream: BoxStream<'static, BeaconEngineMessage<EngineT>>,
|
||||||
hooks: EngineHooks,
|
hooks: EngineHooks,
|
||||||
) -> RethResult<(Self, BeaconConsensusEngineHandle<EngineT>)> {
|
) -> RethResult<(Self, BeaconConsensusEngineHandle<EngineT>)> {
|
||||||
let handle = BeaconConsensusEngineHandle { to_engine };
|
let handle = BeaconConsensusEngineHandle { to_engine };
|
||||||
@ -303,7 +297,7 @@ where
|
|||||||
payload_validator: ExecutionPayloadValidator::new(blockchain.chain_spec()),
|
payload_validator: ExecutionPayloadValidator::new(blockchain.chain_spec()),
|
||||||
blockchain,
|
blockchain,
|
||||||
sync_state_updater,
|
sync_state_updater,
|
||||||
engine_message_rx: UnboundedReceiverStream::new(rx),
|
engine_message_stream,
|
||||||
handle: handle.clone(),
|
handle: handle.clone(),
|
||||||
forkchoice_state_tracker: Default::default(),
|
forkchoice_state_tracker: Default::default(),
|
||||||
payload_builder,
|
payload_builder,
|
||||||
@ -1770,7 +1764,7 @@ where
|
|||||||
//
|
//
|
||||||
// These messages can affect the state of the SyncController and they're also time
|
// These messages can affect the state of the SyncController and they're also time
|
||||||
// sensitive, hence they are polled first.
|
// sensitive, hence they are polled first.
|
||||||
if let Poll::Ready(Some(msg)) = this.engine_message_rx.poll_next_unpin(cx) {
|
if let Poll::Ready(Some(msg)) = this.engine_message_stream.poll_next_unpin(cx) {
|
||||||
match msg {
|
match msg {
|
||||||
BeaconEngineMessage::ForkchoiceUpdated { state, payload_attrs, tx } => {
|
BeaconEngineMessage::ForkchoiceUpdated { state, payload_attrs, tx } => {
|
||||||
this.on_forkchoice_updated(state, payload_attrs, tx);
|
this.on_forkchoice_updated(state, payload_attrs, tx);
|
||||||
|
|||||||
@ -44,6 +44,8 @@ discv5.workspace = true
|
|||||||
|
|
||||||
# async
|
# async
|
||||||
tokio.workspace = true
|
tokio.workspace = true
|
||||||
|
tokio-util.workspace = true
|
||||||
|
pin-project.workspace = true
|
||||||
|
|
||||||
# metrics
|
# metrics
|
||||||
metrics-exporter-prometheus = "0.12.1"
|
metrics-exporter-prometheus = "0.12.1"
|
||||||
|
|||||||
@ -1,5 +1,6 @@
|
|||||||
//! Stores engine API messages to disk for later inspection and replay.
|
//! Stores engine API messages to disk for later inspection and replay.
|
||||||
|
|
||||||
|
use futures::{Stream, StreamExt};
|
||||||
use reth_beacon_consensus::BeaconEngineMessage;
|
use reth_beacon_consensus::BeaconEngineMessage;
|
||||||
use reth_engine_primitives::EngineTypes;
|
use reth_engine_primitives::EngineTypes;
|
||||||
use reth_primitives::fs;
|
use reth_primitives::fs;
|
||||||
@ -8,8 +9,13 @@ use reth_rpc_types::{
|
|||||||
ExecutionPayload,
|
ExecutionPayload,
|
||||||
};
|
};
|
||||||
use serde::{Deserialize, Serialize};
|
use serde::{Deserialize, Serialize};
|
||||||
use std::{collections::BTreeMap, path::PathBuf, time::SystemTime};
|
use std::{
|
||||||
use tokio::sync::mpsc::{UnboundedReceiver, UnboundedSender};
|
collections::BTreeMap,
|
||||||
|
path::PathBuf,
|
||||||
|
pin::Pin,
|
||||||
|
task::{ready, Context, Poll},
|
||||||
|
time::SystemTime,
|
||||||
|
};
|
||||||
use tracing::*;
|
use tracing::*;
|
||||||
|
|
||||||
/// A message from the engine API that has been stored to disk.
|
/// A message from the engine API that has been stored to disk.
|
||||||
@ -34,13 +40,13 @@ pub enum StoredEngineApiMessage<Attributes> {
|
|||||||
|
|
||||||
/// This can read and write engine API messages in a specific directory.
|
/// This can read and write engine API messages in a specific directory.
|
||||||
#[derive(Debug)]
|
#[derive(Debug)]
|
||||||
pub struct EngineApiStore {
|
pub struct EngineMessageStore {
|
||||||
/// The path to the directory that stores the engine API messages.
|
/// The path to the directory that stores the engine API messages.
|
||||||
path: PathBuf,
|
path: PathBuf,
|
||||||
}
|
}
|
||||||
|
|
||||||
impl EngineApiStore {
|
impl EngineMessageStore {
|
||||||
/// Creates a new [EngineApiStore] at the given path.
|
/// Creates a new [EngineMessageStore] at the given path.
|
||||||
///
|
///
|
||||||
/// The path is expected to be a directory, where individual message JSON files will be stored.
|
/// The path is expected to be a directory, where individual message JSON files will be stored.
|
||||||
pub fn new(path: PathBuf) -> Self {
|
pub fn new(path: PathBuf) -> Self {
|
||||||
@ -108,22 +114,42 @@ impl EngineApiStore {
|
|||||||
}
|
}
|
||||||
Ok(filenames_by_ts.into_iter().flat_map(|(_, paths)| paths))
|
Ok(filenames_by_ts.into_iter().flat_map(|(_, paths)| paths))
|
||||||
}
|
}
|
||||||
|
}
|
||||||
|
|
||||||
/// Intercepts an incoming engine API message, storing it to disk and forwarding it to the
|
/// A wrapper stream that stores Engine API messages in
|
||||||
/// engine channel.
|
/// the specified directory.
|
||||||
pub async fn intercept<Engine>(
|
#[derive(Debug)]
|
||||||
self,
|
#[pin_project::pin_project]
|
||||||
mut rx: UnboundedReceiver<BeaconEngineMessage<Engine>>,
|
pub struct EngineStoreStream<S> {
|
||||||
to_engine: UnboundedSender<BeaconEngineMessage<Engine>>,
|
/// Inner message stream.
|
||||||
) where
|
#[pin]
|
||||||
Engine: EngineTypes,
|
stream: S,
|
||||||
BeaconEngineMessage<Engine>: std::fmt::Debug,
|
/// Engine message store.
|
||||||
{
|
store: EngineMessageStore,
|
||||||
while let Some(msg) = rx.recv().await {
|
}
|
||||||
if let Err(error) = self.on_message(&msg, SystemTime::now()) {
|
|
||||||
error!(target: "engine::intercept", ?msg, %error, "Error handling Engine API message");
|
impl<S> EngineStoreStream<S> {
|
||||||
}
|
/// Create new engine store stream wrapper.
|
||||||
let _ = to_engine.send(msg);
|
pub fn new(stream: S, path: PathBuf) -> Self {
|
||||||
}
|
Self { stream, store: EngineMessageStore::new(path) }
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
impl<Engine, S> Stream for EngineStoreStream<S>
|
||||||
|
where
|
||||||
|
Engine: EngineTypes,
|
||||||
|
S: Stream<Item = BeaconEngineMessage<Engine>>,
|
||||||
|
{
|
||||||
|
type Item = S::Item;
|
||||||
|
|
||||||
|
fn poll_next(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
|
||||||
|
let mut this = self.project();
|
||||||
|
let next = ready!(this.stream.poll_next_unpin(cx));
|
||||||
|
if let Some(msg) = &next {
|
||||||
|
if let Err(error) = this.store.on_message(msg, SystemTime::now()) {
|
||||||
|
error!(target: "engine::intercept", ?msg, %error, "Error handling Engine API message");
|
||||||
|
}
|
||||||
|
}
|
||||||
|
Poll::Ready(next)
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
71
crates/node-core/src/engine/mod.rs
Normal file
71
crates/node-core/src/engine/mod.rs
Normal file
@ -0,0 +1,71 @@
|
|||||||
|
//! Collection of various stream utilities for consensus engine.
|
||||||
|
|
||||||
|
use futures::Stream;
|
||||||
|
use reth_beacon_consensus::BeaconEngineMessage;
|
||||||
|
use reth_engine_primitives::EngineTypes;
|
||||||
|
use std::path::PathBuf;
|
||||||
|
use tokio_util::either::Either;
|
||||||
|
|
||||||
|
pub mod engine_store;
|
||||||
|
use engine_store::EngineStoreStream;
|
||||||
|
|
||||||
|
pub mod skip_fcu;
|
||||||
|
use skip_fcu::EngineSkipFcu;
|
||||||
|
|
||||||
|
/// The collection of stream extensions for engine API message stream.
|
||||||
|
pub trait EngineMessageStreamExt<Engine: EngineTypes>:
|
||||||
|
Stream<Item = BeaconEngineMessage<Engine>>
|
||||||
|
{
|
||||||
|
/// Skips the specified number of [BeaconEngineMessage::ForkchoiceUpdated] messages from the
|
||||||
|
/// engine message stream.
|
||||||
|
fn skip_fcu(self, count: usize) -> EngineSkipFcu<Self>
|
||||||
|
where
|
||||||
|
Self: Sized,
|
||||||
|
{
|
||||||
|
EngineSkipFcu::new(self, count)
|
||||||
|
}
|
||||||
|
|
||||||
|
/// If the count is [Some], returns the stream that skips the specified number of
|
||||||
|
/// [BeaconEngineMessage::ForkchoiceUpdated] messages. Otherwise, returns `Self`.
|
||||||
|
fn maybe_skip_fcu(self, maybe_count: Option<usize>) -> Either<EngineSkipFcu<Self>, Self>
|
||||||
|
where
|
||||||
|
Self: Sized,
|
||||||
|
{
|
||||||
|
if let Some(count) = maybe_count {
|
||||||
|
Either::Left(self.skip_fcu(count))
|
||||||
|
} else {
|
||||||
|
Either::Right(self)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
/// Stores engine messages at the specified location.
|
||||||
|
fn store_messages(self, path: PathBuf) -> EngineStoreStream<Self>
|
||||||
|
where
|
||||||
|
Self: Sized,
|
||||||
|
{
|
||||||
|
EngineStoreStream::new(self, path)
|
||||||
|
}
|
||||||
|
|
||||||
|
/// If the path is [Some], returns the stream that stores engine messages at the specified
|
||||||
|
/// location. Otherwise, returns `Self`.
|
||||||
|
fn maybe_store_messages(
|
||||||
|
self,
|
||||||
|
maybe_path: Option<PathBuf>,
|
||||||
|
) -> Either<EngineStoreStream<Self>, Self>
|
||||||
|
where
|
||||||
|
Self: Sized,
|
||||||
|
{
|
||||||
|
if let Some(path) = maybe_path {
|
||||||
|
Either::Left(self.store_messages(path))
|
||||||
|
} else {
|
||||||
|
Either::Right(self)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
impl<Engine, T> EngineMessageStreamExt<Engine> for T
|
||||||
|
where
|
||||||
|
Engine: EngineTypes,
|
||||||
|
T: Stream<Item = BeaconEngineMessage<Engine>>,
|
||||||
|
{
|
||||||
|
}
|
||||||
64
crates/node-core/src/engine/skip_fcu.rs
Normal file
64
crates/node-core/src/engine/skip_fcu.rs
Normal file
@ -0,0 +1,64 @@
|
|||||||
|
//! Stores engine API messages to disk for later inspection and replay.
|
||||||
|
|
||||||
|
use futures::{Stream, StreamExt};
|
||||||
|
use reth_beacon_consensus::{BeaconEngineMessage, OnForkChoiceUpdated};
|
||||||
|
use reth_engine_primitives::EngineTypes;
|
||||||
|
use std::{
|
||||||
|
pin::Pin,
|
||||||
|
task::{ready, Context, Poll},
|
||||||
|
};
|
||||||
|
|
||||||
|
/// Engine API stream wrapper that skips the specified number of forkchoice updated messages.
|
||||||
|
#[derive(Debug)]
|
||||||
|
#[pin_project::pin_project]
|
||||||
|
pub struct EngineSkipFcu<S> {
|
||||||
|
#[pin]
|
||||||
|
stream: S,
|
||||||
|
/// The number of FCUs to skip.
|
||||||
|
threshold: usize,
|
||||||
|
/// Current count of skipped FCUs.
|
||||||
|
skipped: usize,
|
||||||
|
}
|
||||||
|
|
||||||
|
impl<S> EngineSkipFcu<S> {
|
||||||
|
/// Creates new [EngineSkipFcu] stream wrapper.
|
||||||
|
pub fn new(stream: S, threshold: usize) -> Self {
|
||||||
|
Self {
|
||||||
|
stream,
|
||||||
|
threshold,
|
||||||
|
// Start with `threshold` so that the first FCU goes through.
|
||||||
|
skipped: threshold,
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
impl<Engine, S> Stream for EngineSkipFcu<S>
|
||||||
|
where
|
||||||
|
Engine: EngineTypes,
|
||||||
|
S: Stream<Item = BeaconEngineMessage<Engine>>,
|
||||||
|
{
|
||||||
|
type Item = S::Item;
|
||||||
|
|
||||||
|
fn poll_next(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
|
||||||
|
let mut this = self.project();
|
||||||
|
|
||||||
|
loop {
|
||||||
|
let next = ready!(this.stream.poll_next_unpin(cx));
|
||||||
|
let item = match next {
|
||||||
|
Some(BeaconEngineMessage::ForkchoiceUpdated { state, payload_attrs, tx }) => {
|
||||||
|
if this.skipped < this.threshold {
|
||||||
|
*this.skipped += 1;
|
||||||
|
tracing::warn!(target: "engine::intercept", ?state, ?payload_attrs, threshold=this.threshold, skipped=this.skipped, "Skipping FCU");
|
||||||
|
let _ = tx.send(Ok(OnForkChoiceUpdated::syncing()));
|
||||||
|
continue
|
||||||
|
} else {
|
||||||
|
*this.skipped = 0;
|
||||||
|
Some(BeaconEngineMessage::ForkchoiceUpdated { state, payload_attrs, tx })
|
||||||
|
}
|
||||||
|
}
|
||||||
|
next => next,
|
||||||
|
};
|
||||||
|
return Poll::Ready(item)
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
@ -1,55 +0,0 @@
|
|||||||
//! Stores engine API messages to disk for later inspection and replay.
|
|
||||||
|
|
||||||
use reth_beacon_consensus::{BeaconEngineMessage, OnForkChoiceUpdated};
|
|
||||||
use reth_engine_primitives::EngineTypes;
|
|
||||||
use tokio::sync::mpsc::{UnboundedReceiver, UnboundedSender};
|
|
||||||
|
|
||||||
/// Intercept Engine API message and skip FCUs.
|
|
||||||
#[derive(Debug)]
|
|
||||||
pub struct EngineApiSkipFcu {
|
|
||||||
/// The number of FCUs to skip.
|
|
||||||
threshold: usize,
|
|
||||||
/// Current count of skipped FCUs.
|
|
||||||
skipped: usize,
|
|
||||||
}
|
|
||||||
|
|
||||||
impl EngineApiSkipFcu {
|
|
||||||
/// Creates new [EngineApiSkipFcu] interceptor.
|
|
||||||
pub fn new(threshold: usize) -> Self {
|
|
||||||
Self {
|
|
||||||
threshold,
|
|
||||||
// Start with `threshold` so that the first FCU goes through.
|
|
||||||
skipped: threshold,
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
/// Intercepts an incoming engine API message, skips FCU or forwards it
|
|
||||||
/// to the engine depending on current number of skipped FCUs.
|
|
||||||
pub async fn intercept<Engine>(
|
|
||||||
mut self,
|
|
||||||
mut rx: UnboundedReceiver<BeaconEngineMessage<Engine>>,
|
|
||||||
to_engine: UnboundedSender<BeaconEngineMessage<Engine>>,
|
|
||||||
) where
|
|
||||||
Engine: EngineTypes,
|
|
||||||
BeaconEngineMessage<Engine>: std::fmt::Debug,
|
|
||||||
{
|
|
||||||
while let Some(msg) = rx.recv().await {
|
|
||||||
if let BeaconEngineMessage::ForkchoiceUpdated { state, payload_attrs, tx } = msg {
|
|
||||||
if self.skipped < self.threshold {
|
|
||||||
self.skipped += 1;
|
|
||||||
tracing::warn!(target: "engine::intercept", ?state, ?payload_attrs, threshold=self.threshold, skipped=self.skipped, "Skipping FCU");
|
|
||||||
let _ = tx.send(Ok(OnForkChoiceUpdated::syncing()));
|
|
||||||
} else {
|
|
||||||
self.skipped = 0;
|
|
||||||
let _ = to_engine.send(BeaconEngineMessage::ForkchoiceUpdated {
|
|
||||||
state,
|
|
||||||
payload_attrs,
|
|
||||||
tx,
|
|
||||||
});
|
|
||||||
}
|
|
||||||
} else {
|
|
||||||
let _ = to_engine.send(msg);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
@ -11,8 +11,7 @@
|
|||||||
pub mod args;
|
pub mod args;
|
||||||
pub mod cli;
|
pub mod cli;
|
||||||
pub mod dirs;
|
pub mod dirs;
|
||||||
pub mod engine_api_store;
|
pub mod engine;
|
||||||
pub mod engine_skip_fcu;
|
|
||||||
pub mod exit;
|
pub mod exit;
|
||||||
pub mod init;
|
pub mod init;
|
||||||
pub mod metrics;
|
pub mod metrics;
|
||||||
|
|||||||
@ -47,6 +47,7 @@ tokio = { workspace = true, features = [
|
|||||||
"time",
|
"time",
|
||||||
"rt-multi-thread",
|
"rt-multi-thread",
|
||||||
] }
|
] }
|
||||||
|
tokio-stream.workspace = true
|
||||||
|
|
||||||
## misc
|
## misc
|
||||||
aquamarine.workspace = true
|
aquamarine.workspace = true
|
||||||
|
|||||||
@ -14,7 +14,8 @@ use reth_beacon_consensus::{
|
|||||||
BeaconConsensus, BeaconConsensusEngine,
|
BeaconConsensus, BeaconConsensusEngine,
|
||||||
};
|
};
|
||||||
use reth_blockchain_tree::{
|
use reth_blockchain_tree::{
|
||||||
BlockchainTree, BlockchainTreeConfig, ShareableBlockchainTree, TreeExternals,
|
noop::NoopBlockchainTree, BlockchainTree, BlockchainTreeConfig, ShareableBlockchainTree,
|
||||||
|
TreeExternals,
|
||||||
};
|
};
|
||||||
use reth_consensus::Consensus;
|
use reth_consensus::Consensus;
|
||||||
use reth_exex::{ExExContext, ExExHandle, ExExManager, ExExManagerHandle};
|
use reth_exex::{ExExContext, ExExHandle, ExExManager, ExExManagerHandle};
|
||||||
@ -23,8 +24,7 @@ use reth_network::NetworkEvents;
|
|||||||
use reth_node_api::{FullNodeComponents, FullNodeTypes};
|
use reth_node_api::{FullNodeComponents, FullNodeTypes};
|
||||||
use reth_node_core::{
|
use reth_node_core::{
|
||||||
dirs::{ChainPath, DataDirPath},
|
dirs::{ChainPath, DataDirPath},
|
||||||
engine_api_store::EngineApiStore,
|
engine::EngineMessageStreamExt,
|
||||||
engine_skip_fcu::EngineApiSkipFcu,
|
|
||||||
exit::NodeExitFuture,
|
exit::NodeExitFuture,
|
||||||
};
|
};
|
||||||
use reth_node_events::{cl::ConsensusLayerHealthEvents, node};
|
use reth_node_events::{cl::ConsensusLayerHealthEvents, node};
|
||||||
@ -37,10 +37,10 @@ use reth_tracing::tracing::{debug, info};
|
|||||||
use reth_transaction_pool::TransactionPool;
|
use reth_transaction_pool::TransactionPool;
|
||||||
use std::{future::Future, sync::Arc};
|
use std::{future::Future, sync::Arc};
|
||||||
use tokio::sync::{mpsc::unbounded_channel, oneshot};
|
use tokio::sync::{mpsc::unbounded_channel, oneshot};
|
||||||
|
use tokio_stream::wrappers::UnboundedReceiverStream;
|
||||||
|
|
||||||
pub mod common;
|
pub mod common;
|
||||||
pub use common::LaunchContext;
|
pub use common::LaunchContext;
|
||||||
use reth_blockchain_tree::noop::NoopBlockchainTree;
|
|
||||||
|
|
||||||
/// A general purpose trait that launches a new node of any kind.
|
/// A general purpose trait that launches a new node of any kind.
|
||||||
///
|
///
|
||||||
@ -261,29 +261,15 @@ where
|
|||||||
|
|
||||||
// create pipeline
|
// create pipeline
|
||||||
let network_client = node_adapter.network().fetch_client().await?;
|
let network_client = node_adapter.network().fetch_client().await?;
|
||||||
let (consensus_engine_tx, mut consensus_engine_rx) = unbounded_channel();
|
let (consensus_engine_tx, consensus_engine_rx) = unbounded_channel();
|
||||||
|
|
||||||
if let Some(skip_fcu_threshold) = ctx.node_config().debug.skip_fcu {
|
let node_config = ctx.node_config();
|
||||||
debug!(target: "reth::cli", "spawning skip FCU task");
|
let consensus_engine_stream = UnboundedReceiverStream::from(consensus_engine_rx)
|
||||||
let (skip_fcu_tx, skip_fcu_rx) = unbounded_channel();
|
.maybe_skip_fcu(node_config.debug.skip_fcu)
|
||||||
let engine_skip_fcu = EngineApiSkipFcu::new(skip_fcu_threshold);
|
// Store messages _after_ skipping messages so that `replay-engine` command
|
||||||
ctx.task_executor().spawn_critical(
|
// would replay the exact same messages that were observed by the engine
|
||||||
"skip FCU interceptor",
|
// during this run.
|
||||||
engine_skip_fcu.intercept(consensus_engine_rx, skip_fcu_tx),
|
.maybe_store_messages(node_config.debug.engine_api_store.clone());
|
||||||
);
|
|
||||||
consensus_engine_rx = skip_fcu_rx;
|
|
||||||
}
|
|
||||||
|
|
||||||
if let Some(store_path) = ctx.node_config().debug.engine_api_store.clone() {
|
|
||||||
debug!(target: "reth::cli", "spawning engine API store");
|
|
||||||
let (engine_intercept_tx, engine_intercept_rx) = unbounded_channel();
|
|
||||||
let engine_api_store = EngineApiStore::new(store_path);
|
|
||||||
ctx.task_executor().spawn_critical(
|
|
||||||
"engine api interceptor",
|
|
||||||
engine_api_store.intercept(consensus_engine_rx, engine_intercept_tx),
|
|
||||||
);
|
|
||||||
consensus_engine_rx = engine_intercept_rx;
|
|
||||||
};
|
|
||||||
|
|
||||||
let max_block = ctx.max_block(network_client.clone()).await?;
|
let max_block = ctx.max_block(network_client.clone()).await?;
|
||||||
let mut hooks = EngineHooks::new();
|
let mut hooks = EngineHooks::new();
|
||||||
@ -303,8 +289,7 @@ where
|
|||||||
info!(target: "reth::cli", "Starting Reth in dev mode");
|
info!(target: "reth::cli", "Starting Reth in dev mode");
|
||||||
|
|
||||||
for (idx, (address, alloc)) in ctx.chain_spec().genesis.alloc.iter().enumerate() {
|
for (idx, (address, alloc)) in ctx.chain_spec().genesis.alloc.iter().enumerate() {
|
||||||
info!(target: "reth::cli", "Allocated Genesis Account: {:02}. {} ({} ETH)", idx,
|
info!(target: "reth::cli", "Allocated Genesis Account: {:02}. {} ({} ETH)", idx, address.to_string(), format_ether(alloc.balance));
|
||||||
address.to_string(), format_ether(alloc.balance));
|
|
||||||
}
|
}
|
||||||
|
|
||||||
// install auto-seal
|
// install auto-seal
|
||||||
@ -395,7 +380,7 @@ address.to_string(), format_ether(alloc.balance));
|
|||||||
initial_target,
|
initial_target,
|
||||||
reth_beacon_consensus::MIN_BLOCKS_FOR_PIPELINE_RUN,
|
reth_beacon_consensus::MIN_BLOCKS_FOR_PIPELINE_RUN,
|
||||||
consensus_engine_tx,
|
consensus_engine_tx,
|
||||||
consensus_engine_rx,
|
Box::pin(consensus_engine_stream),
|
||||||
hooks,
|
hooks,
|
||||||
)?;
|
)?;
|
||||||
info!(target: "reth::cli", "Consensus engine initialized");
|
info!(target: "reth::cli", "Consensus engine initialized");
|
||||||
|
|||||||
Reference in New Issue
Block a user