mirror of
https://github.com/hl-archive-node/nanoreth.git
synced 2025-12-06 10:59:55 +00:00
perf: warm transactions in parallel (#13759)
This commit is contained in:
1
Cargo.lock
generated
1
Cargo.lock
generated
@ -7227,6 +7227,7 @@ dependencies = [
|
||||
"reth-ethereum-engine-primitives",
|
||||
"reth-ethereum-primitives",
|
||||
"reth-evm",
|
||||
"reth-evm-ethereum",
|
||||
"reth-exex-types",
|
||||
"reth-metrics",
|
||||
"reth-network-p2p",
|
||||
|
||||
@ -9,9 +9,7 @@ use alloy_rlp::Decodable;
|
||||
use alloy_rpc_types::engine::{BlobsBundleV1, PayloadAttributes};
|
||||
use clap::Parser;
|
||||
use eyre::Context;
|
||||
use reth_basic_payload_builder::{
|
||||
BuildArguments, BuildOutcome, Cancelled, PayloadBuilder, PayloadConfig,
|
||||
};
|
||||
use reth_basic_payload_builder::{BuildArguments, BuildOutcome, PayloadBuilder, PayloadConfig};
|
||||
use reth_chainspec::ChainSpec;
|
||||
use reth_cli::chainspec::ChainSpecParser;
|
||||
use reth_cli_commands::common::{AccessRights, CliNodeTypes, Environment, EnvironmentArgs};
|
||||
@ -34,7 +32,10 @@ use reth_provider::{
|
||||
BlockHashReader, BlockReader, BlockWriter, ChainSpecProvider, ProviderFactory,
|
||||
StageCheckpointReader, StateProviderFactory,
|
||||
};
|
||||
use reth_revm::{cached::CachedReads, database::StateProviderDatabase, primitives::KzgSettings};
|
||||
use reth_revm::{
|
||||
cached::CachedReads, cancelled::Cancelled, database::StateProviderDatabase,
|
||||
primitives::KzgSettings,
|
||||
};
|
||||
use reth_stages::StageId;
|
||||
use reth_transaction_pool::{
|
||||
blobstore::InMemoryBlobStore, BlobStore, EthPooledTransaction, PoolConfig, TransactionOrigin,
|
||||
|
||||
@ -29,8 +29,8 @@ use reth_engine_tree::{
|
||||
persistence::PersistenceHandle,
|
||||
tree::{EngineApiTreeHandler, InvalidBlockHook, TreeConfig},
|
||||
};
|
||||
use reth_evm::execute::BlockExecutorProvider;
|
||||
use reth_node_types::BlockTy;
|
||||
use reth_evm::{execute::BlockExecutorProvider, ConfigureEvm};
|
||||
use reth_node_types::{BlockTy, HeaderTy, TxTy};
|
||||
use reth_payload_builder::PayloadBuilderHandle;
|
||||
use reth_payload_primitives::{PayloadAttributesBuilder, PayloadTypes};
|
||||
use reth_provider::{
|
||||
@ -65,7 +65,7 @@ where
|
||||
{
|
||||
/// Constructor for [`LocalEngineService`].
|
||||
#[allow(clippy::too_many_arguments)]
|
||||
pub fn new<B, V>(
|
||||
pub fn new<B, V, C>(
|
||||
consensus: Arc<dyn FullConsensus<N::Primitives, Error = ConsensusError>>,
|
||||
executor_factory: impl BlockExecutorProvider<Primitives = N::Primitives>,
|
||||
provider: ProviderFactory<N>,
|
||||
@ -80,10 +80,12 @@ where
|
||||
from_engine: EngineMessageStream<N::Engine>,
|
||||
mode: MiningMode,
|
||||
payload_attributes_builder: B,
|
||||
evm_config: C,
|
||||
) -> Self
|
||||
where
|
||||
B: PayloadAttributesBuilder<<N::Engine as PayloadTypes>::PayloadAttributes>,
|
||||
V: EngineValidator<N::Engine, Block = BlockTy<N>>,
|
||||
C: ConfigureEvm<Header = HeaderTy<N>, Transaction = TxTy<N>>,
|
||||
{
|
||||
let chain_spec = provider.chain_spec();
|
||||
let engine_kind =
|
||||
@ -93,18 +95,20 @@ where
|
||||
PersistenceHandle::<N::Primitives>::spawn_service(provider, pruner, sync_metrics_tx);
|
||||
let canonical_in_memory_state = blockchain_db.canonical_in_memory_state();
|
||||
|
||||
let (to_tree_tx, from_tree) = EngineApiTreeHandler::<N::Primitives, _, _, _, _>::spawn_new(
|
||||
blockchain_db.clone(),
|
||||
executor_factory,
|
||||
consensus,
|
||||
payload_validator,
|
||||
persistence_handle,
|
||||
payload_builder.clone(),
|
||||
canonical_in_memory_state,
|
||||
tree_config,
|
||||
invalid_block_hook,
|
||||
engine_kind,
|
||||
);
|
||||
let (to_tree_tx, from_tree) =
|
||||
EngineApiTreeHandler::<N::Primitives, _, _, _, _, _>::spawn_new(
|
||||
blockchain_db.clone(),
|
||||
executor_factory,
|
||||
consensus,
|
||||
payload_validator,
|
||||
persistence_handle,
|
||||
payload_builder.clone(),
|
||||
canonical_in_memory_state,
|
||||
tree_config,
|
||||
invalid_block_hook,
|
||||
engine_kind,
|
||||
evm_config,
|
||||
);
|
||||
|
||||
let handler = EngineApiRequestHandler::new(to_tree_tx, from_tree);
|
||||
|
||||
|
||||
@ -14,9 +14,9 @@ pub use reth_engine_tree::{
|
||||
chain::{ChainEvent, ChainOrchestrator},
|
||||
engine::EngineApiEvent,
|
||||
};
|
||||
use reth_evm::execute::BlockExecutorProvider;
|
||||
use reth_evm::{execute::BlockExecutorProvider, ConfigureEvm};
|
||||
use reth_network_p2p::BlockClient;
|
||||
use reth_node_types::{BlockTy, NodeTypes, NodeTypesWithEngine};
|
||||
use reth_node_types::{BlockTy, HeaderTy, NodeTypes, NodeTypesWithEngine, TxTy};
|
||||
use reth_payload_builder::PayloadBuilderHandle;
|
||||
use reth_primitives::EthPrimitives;
|
||||
use reth_provider::{
|
||||
@ -73,7 +73,7 @@ where
|
||||
{
|
||||
/// Constructor for `EngineService`.
|
||||
#[allow(clippy::too_many_arguments)]
|
||||
pub fn new<V>(
|
||||
pub fn new<V, C>(
|
||||
consensus: Arc<dyn FullConsensus<N::Primitives, Error = ConsensusError>>,
|
||||
executor_factory: E,
|
||||
chain_spec: Arc<N::ChainSpec>,
|
||||
@ -89,9 +89,11 @@ where
|
||||
tree_config: TreeConfig,
|
||||
invalid_block_hook: Box<dyn InvalidBlockHook<N::Primitives>>,
|
||||
sync_metrics_tx: MetricEventsSender,
|
||||
evm_config: C,
|
||||
) -> Self
|
||||
where
|
||||
V: EngineValidator<N::Engine, Block = BlockTy<N>>,
|
||||
C: ConfigureEvm<Header = HeaderTy<N>, Transaction = TxTy<N>>,
|
||||
{
|
||||
let engine_kind =
|
||||
if chain_spec.is_optimism() { EngineApiKind::OpStack } else { EngineApiKind::Ethereum };
|
||||
@ -103,18 +105,20 @@ where
|
||||
|
||||
let canonical_in_memory_state = blockchain_db.canonical_in_memory_state();
|
||||
|
||||
let (to_tree_tx, from_tree) = EngineApiTreeHandler::<N::Primitives, _, _, _, _>::spawn_new(
|
||||
blockchain_db,
|
||||
executor_factory,
|
||||
consensus,
|
||||
payload_validator,
|
||||
persistence_handle,
|
||||
payload_builder,
|
||||
canonical_in_memory_state,
|
||||
tree_config,
|
||||
invalid_block_hook,
|
||||
engine_kind,
|
||||
);
|
||||
let (to_tree_tx, from_tree) =
|
||||
EngineApiTreeHandler::<N::Primitives, _, _, _, _, _>::spawn_new(
|
||||
blockchain_db,
|
||||
executor_factory,
|
||||
consensus,
|
||||
payload_validator,
|
||||
persistence_handle,
|
||||
payload_builder,
|
||||
canonical_in_memory_state,
|
||||
tree_config,
|
||||
invalid_block_hook,
|
||||
engine_kind,
|
||||
evm_config,
|
||||
);
|
||||
|
||||
let engine_handler = EngineApiRequestHandler::new(to_tree_tx, from_tree);
|
||||
let handler = EngineHandler::new(engine_handler, downloader, incoming_requests);
|
||||
@ -160,7 +164,7 @@ mod tests {
|
||||
use reth_engine_tree::{test_utils::TestPipelineBuilder, tree::NoopInvalidBlockHook};
|
||||
use reth_ethereum_consensus::EthBeaconConsensus;
|
||||
use reth_ethereum_engine_primitives::{EthEngineTypes, EthereumEngineValidator};
|
||||
use reth_evm_ethereum::execute::EthExecutorProvider;
|
||||
use reth_evm_ethereum::{execute::EthExecutorProvider, EthEvmConfig};
|
||||
use reth_exex_types::FinishedExExHeight;
|
||||
use reth_network_p2p::test_utils::TestFullBlockClient;
|
||||
use reth_primitives::SealedHeader;
|
||||
@ -200,6 +204,7 @@ mod tests {
|
||||
let engine_payload_validator = EthereumEngineValidator::new(chain_spec.clone());
|
||||
let (_tx, rx) = watch::channel(FinishedExExHeight::NoExExs);
|
||||
let pruner = Pruner::new_with_factory(provider_factory.clone(), vec![], 0, 0, None, rx);
|
||||
let evm_config = EthEvmConfig::new(chain_spec.clone());
|
||||
|
||||
let (sync_metrics_tx, _sync_metrics_rx) = unbounded_channel();
|
||||
let (tx, _rx) = unbounded_channel();
|
||||
@ -219,6 +224,7 @@ mod tests {
|
||||
TreeConfig::default(),
|
||||
Box::new(NoopInvalidBlockHook::default()),
|
||||
sync_metrics_tx,
|
||||
evm_config,
|
||||
);
|
||||
}
|
||||
}
|
||||
|
||||
@ -72,6 +72,7 @@ reth-chain-state = { workspace = true, features = ["test-utils"] }
|
||||
reth-chainspec.workspace = true
|
||||
reth-ethereum-engine-primitives.workspace = true
|
||||
reth-ethereum-consensus.workspace = true
|
||||
reth-evm-ethereum.workspace = true
|
||||
reth-evm = { workspace = true, features = ["test-utils"] }
|
||||
reth-exex-types.workspace = true
|
||||
reth-network-p2p = { workspace = true, features = ["test-utils"] }
|
||||
|
||||
@ -15,7 +15,7 @@ use reth_trie::{
|
||||
};
|
||||
use revm_primitives::map::DefaultHashBuilder;
|
||||
|
||||
type Cache<K, V> = moka::sync::Cache<K, V, alloy_primitives::map::DefaultHashBuilder>;
|
||||
pub(crate) type Cache<K, V> = moka::sync::Cache<K, V, alloy_primitives::map::DefaultHashBuilder>;
|
||||
|
||||
/// A wrapper of a state provider and a shared cache.
|
||||
pub(crate) struct CachedStateProvider<S> {
|
||||
|
||||
@ -11,13 +11,15 @@ use crate::{
|
||||
use alloy_consensus::BlockHeader;
|
||||
use alloy_eips::BlockNumHash;
|
||||
use alloy_primitives::{
|
||||
map::{HashMap, HashSet},
|
||||
BlockNumber, B256, U256,
|
||||
keccak256,
|
||||
map::{B256Set, HashMap, HashSet},
|
||||
Address, BlockNumber, B256, U256,
|
||||
};
|
||||
use alloy_rpc_types_engine::{
|
||||
ForkchoiceState, PayloadStatus, PayloadStatusEnum, PayloadValidationError,
|
||||
};
|
||||
use block_buffer::BlockBuffer;
|
||||
use cached_state::ProviderCaches;
|
||||
use error::{InsertBlockError, InsertBlockErrorKind, InsertBlockFatalError};
|
||||
use persistence_state::CurrentPersistenceAction;
|
||||
use reth_chain_state::{
|
||||
@ -35,26 +37,32 @@ use reth_ethereum_primitives::EthPrimitives;
|
||||
use reth_evm::{
|
||||
execute::BlockExecutorProvider,
|
||||
system_calls::{NoopHook, OnStateHook},
|
||||
ConfigureEvm, Evm, TransactionEnv,
|
||||
};
|
||||
use reth_payload_builder::PayloadBuilderHandle;
|
||||
use reth_payload_builder_primitives::PayloadBuilder;
|
||||
use reth_payload_primitives::{EngineApiMessageVersion, PayloadBuilderAttributes};
|
||||
use reth_primitives_traits::{
|
||||
Block, GotExpected, NodePrimitives, RecoveredBlock, SealedBlock, SealedHeader,
|
||||
Block, BlockBody, GotExpected, NodePrimitives, RecoveredBlock, SealedBlock, SealedHeader,
|
||||
SignedTransaction,
|
||||
};
|
||||
use reth_provider::{
|
||||
providers::ConsistentDbView, BlockReader, DBProvider, DatabaseProviderFactory,
|
||||
ExecutionOutcome, HashedPostStateProvider, ProviderError, StateCommitmentProvider,
|
||||
StateProviderBox, StateProviderFactory, StateReader, StateRootProvider, TransactionVariant,
|
||||
};
|
||||
use reth_revm::database::StateProviderDatabase;
|
||||
use reth_revm::{cancelled::Cancelled, database::StateProviderDatabase};
|
||||
use reth_stages_api::ControlFlow;
|
||||
use reth_trie::{
|
||||
trie_cursor::InMemoryTrieCursorFactory, updates::TrieUpdates, HashedPostState, TrieInput,
|
||||
trie_cursor::InMemoryTrieCursorFactory, updates::TrieUpdates, HashedPostState,
|
||||
MultiProofTargets, TrieInput,
|
||||
};
|
||||
use reth_trie_db::DatabaseTrieCursorFactory;
|
||||
use reth_trie_parallel::root::{ParallelStateRoot, ParallelStateRootError};
|
||||
use root::{StateRootComputeOutcome, StateRootConfig, StateRootHandle, StateRootTask};
|
||||
use revm_primitives::ResultAndState;
|
||||
use root::{
|
||||
StateRootComputeOutcome, StateRootConfig, StateRootHandle, StateRootMessage, StateRootTask,
|
||||
};
|
||||
use std::{
|
||||
cmp::Ordering,
|
||||
collections::{btree_map, hash_map, BTreeMap, VecDeque},
|
||||
@ -518,13 +526,14 @@ pub enum TreeAction {
|
||||
///
|
||||
/// This type is responsible for processing engine API requests, maintaining the canonical state and
|
||||
/// emitting events.
|
||||
pub struct EngineApiTreeHandler<N, P, E, T, V>
|
||||
pub struct EngineApiTreeHandler<N, P, E, T, V, C>
|
||||
where
|
||||
N: NodePrimitives,
|
||||
T: EngineTypes,
|
||||
{
|
||||
provider: P,
|
||||
executor_provider: E,
|
||||
evm_config: C,
|
||||
consensus: Arc<dyn FullConsensus<N, Error = ConsensusError>>,
|
||||
payload_validator: V,
|
||||
/// Keeps track of internals such as executed and buffered blocks.
|
||||
@ -562,18 +571,19 @@ where
|
||||
invalid_block_hook: Box<dyn InvalidBlockHook<N>>,
|
||||
/// The engine API variant of this handler
|
||||
engine_kind: EngineApiKind,
|
||||
/// state root task thread pool
|
||||
state_root_task_pool: Arc<rayon::ThreadPool>,
|
||||
/// Thread pool used for the state root task and prewarming
|
||||
thread_pool: Arc<rayon::ThreadPool>,
|
||||
}
|
||||
|
||||
impl<N, P: Debug, E: Debug, T: EngineTypes + Debug, V: Debug> std::fmt::Debug
|
||||
for EngineApiTreeHandler<N, P, E, T, V>
|
||||
impl<N, P: Debug, E: Debug, T: EngineTypes + Debug, V: Debug, C: Debug> std::fmt::Debug
|
||||
for EngineApiTreeHandler<N, P, E, T, V, C>
|
||||
where
|
||||
N: NodePrimitives,
|
||||
{
|
||||
fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
|
||||
f.debug_struct("EngineApiTreeHandler")
|
||||
.field("provider", &self.provider)
|
||||
.field("evm_config", &self.evm_config)
|
||||
.field("executor_provider", &self.executor_provider)
|
||||
.field("consensus", &self.consensus)
|
||||
.field("payload_validator", &self.payload_validator)
|
||||
@ -592,7 +602,7 @@ where
|
||||
}
|
||||
}
|
||||
|
||||
impl<N, P, E, T, V> EngineApiTreeHandler<N, P, E, T, V>
|
||||
impl<N, P, E, T, V, C> EngineApiTreeHandler<N, P, E, T, V, C>
|
||||
where
|
||||
N: NodePrimitives,
|
||||
P: DatabaseProviderFactory
|
||||
@ -606,6 +616,7 @@ where
|
||||
<P as DatabaseProviderFactory>::Provider:
|
||||
BlockReader<Block = N::Block, Header = N::BlockHeader>,
|
||||
E: BlockExecutorProvider<Primitives = N>,
|
||||
C: ConfigureEvm<Header = N::BlockHeader, Transaction = N::SignedTx>,
|
||||
T: EngineTypes,
|
||||
V: EngineValidator<T, Block = N::Block>,
|
||||
{
|
||||
@ -624,12 +635,13 @@ where
|
||||
payload_builder: PayloadBuilderHandle<T>,
|
||||
config: TreeConfig,
|
||||
engine_kind: EngineApiKind,
|
||||
evm_config: C,
|
||||
) -> Self {
|
||||
let (incoming_tx, incoming) = std::sync::mpsc::channel();
|
||||
|
||||
let num_threads = root::thread_pool_size();
|
||||
|
||||
let state_root_task_pool = Arc::new(
|
||||
let thread_pool = Arc::new(
|
||||
rayon::ThreadPoolBuilder::new()
|
||||
.num_threads(num_threads)
|
||||
.thread_name(|i| format!("srt-worker-{}", i))
|
||||
@ -640,6 +652,7 @@ where
|
||||
Self {
|
||||
provider,
|
||||
executor_provider,
|
||||
evm_config,
|
||||
consensus,
|
||||
payload_validator,
|
||||
incoming,
|
||||
@ -655,7 +668,7 @@ where
|
||||
incoming_tx,
|
||||
invalid_block_hook: Box::new(NoopInvalidBlockHook),
|
||||
engine_kind,
|
||||
state_root_task_pool,
|
||||
thread_pool,
|
||||
}
|
||||
}
|
||||
|
||||
@ -681,6 +694,7 @@ where
|
||||
config: TreeConfig,
|
||||
invalid_block_hook: Box<dyn InvalidBlockHook<N>>,
|
||||
kind: EngineApiKind,
|
||||
evm_config: C,
|
||||
) -> (Sender<FromEngine<EngineApiRequest<T, N>, N::Block>>, UnboundedReceiver<EngineApiEvent<N>>)
|
||||
{
|
||||
let best_block_number = provider.best_block_number().unwrap_or(0);
|
||||
@ -712,6 +726,7 @@ where
|
||||
payload_builder,
|
||||
config,
|
||||
kind,
|
||||
evm_config,
|
||||
);
|
||||
task.set_invalid_block_hook(invalid_block_hook);
|
||||
let incoming = task.incoming_tx.clone();
|
||||
@ -2347,17 +2362,6 @@ where
|
||||
return Err(e.into())
|
||||
}
|
||||
|
||||
// Use cached state provider before executing, this does nothing currently, will be used in
|
||||
// prewarming
|
||||
let caches = ProviderCacheBuilder::default().build_caches();
|
||||
let cache_metrics = CachedStateMetrics::zeroed();
|
||||
let state_provider =
|
||||
CachedStateProvider::new_with_caches(state_provider, caches, cache_metrics);
|
||||
|
||||
trace!(target: "engine::tree", block=?block_num_hash, "Executing block");
|
||||
|
||||
let executor = self.executor_provider.executor(StateProviderDatabase::new(&state_provider));
|
||||
|
||||
let sealed_block = Arc::new(block.clone_sealed_block());
|
||||
|
||||
// We only run the parallel state root if we are currently persisting blocks that are all
|
||||
@ -2370,7 +2374,10 @@ where
|
||||
let is_descendant_of_persisting_blocks =
|
||||
self.is_descendant_of_persisting_blocks(block.header());
|
||||
|
||||
let (state_root_handle, state_root_task_config, state_hook) =
|
||||
// Atomic bool for letting the prewarm tasks know when to stop
|
||||
let cancel_execution = Cancelled::default();
|
||||
|
||||
let (state_root_handle, state_root_task_config, state_root_sender, state_hook) =
|
||||
if is_descendant_of_persisting_blocks && self.config.use_state_root_task() {
|
||||
let consistent_view = ConsistentDbView::new_with_latest_tip(self.provider.clone())?;
|
||||
|
||||
@ -2397,21 +2404,69 @@ where
|
||||
.state_root_config_duration
|
||||
.set(config_elapsed.as_secs_f64());
|
||||
|
||||
let state_root_task = StateRootTask::new(
|
||||
state_root_config.clone(),
|
||||
self.state_root_task_pool.clone(),
|
||||
);
|
||||
let state_root_task =
|
||||
StateRootTask::new(state_root_config.clone(), self.thread_pool.clone());
|
||||
let state_root_sender = state_root_task.state_root_message_sender();
|
||||
let state_hook = Box::new(state_root_task.state_hook()) as Box<dyn OnStateHook>;
|
||||
(Some(state_root_task.spawn()), Some(state_root_config), state_hook)
|
||||
(
|
||||
Some(state_root_task.spawn()),
|
||||
Some(state_root_config),
|
||||
Some(state_root_sender),
|
||||
state_hook,
|
||||
)
|
||||
} else {
|
||||
(None, None, Box::new(NoopHook::default()) as Box<dyn OnStateHook>)
|
||||
(None, None, None, Box::new(NoopHook::default()) as Box<dyn OnStateHook>)
|
||||
};
|
||||
|
||||
// Use cached state provider before executing, used in execution after prewarming threads
|
||||
// complete
|
||||
let caches = ProviderCacheBuilder::default().build_caches();
|
||||
let cache_metrics = CachedStateMetrics::zeroed();
|
||||
let state_provider = CachedStateProvider::new_with_caches(
|
||||
state_provider,
|
||||
caches.clone(),
|
||||
cache_metrics.clone(),
|
||||
);
|
||||
|
||||
if self.config.use_caching_and_prewarming() {
|
||||
debug!(target: "engine::tree", "Spawning prewarm threads");
|
||||
let prewarm_start = Instant::now();
|
||||
|
||||
// Prewarm transactions
|
||||
for (tx_idx, (tx, sender)) in
|
||||
block.body().transactions().iter().zip(block.senders()).enumerate()
|
||||
{
|
||||
let state_root_sender = state_root_sender.clone();
|
||||
|
||||
let start = Instant::now();
|
||||
self.prewarm_transaction(
|
||||
block.header().clone(),
|
||||
tx.clone(),
|
||||
*sender,
|
||||
caches.clone(),
|
||||
cache_metrics.clone(),
|
||||
state_root_sender,
|
||||
cancel_execution.clone(),
|
||||
)?;
|
||||
let elapsed = start.elapsed();
|
||||
debug!(target: "engine::tree", ?tx_idx, elapsed = ?elapsed, "Spawned transaction prewarm");
|
||||
}
|
||||
|
||||
drop(state_root_sender);
|
||||
let elapsed = prewarm_start.elapsed();
|
||||
debug!(target: "engine::tree", ?elapsed, "Done spawning prewarm threads");
|
||||
}
|
||||
trace!(target: "engine::tree", block=?block_num_hash, "Executing block");
|
||||
|
||||
let executor = self.executor_provider.executor(StateProviderDatabase::new(&state_provider));
|
||||
let execution_start = Instant::now();
|
||||
let output = self.metrics.executor.execute_metered(executor, &block, state_hook)?;
|
||||
let execution_time = execution_start.elapsed();
|
||||
trace!(target: "engine::tree", elapsed = ?execution_time, number=?block_num_hash.number, "Executed block");
|
||||
|
||||
// Ensure that prewarm tasks don't send proof messages after state root sender is dropped
|
||||
drop(cancel_execution);
|
||||
|
||||
if let Err(err) = self.consensus.validate_block_post_execution(
|
||||
&block,
|
||||
PostExecutionInput::new(&output.receipts, &output.requests),
|
||||
@ -2573,6 +2628,89 @@ where
|
||||
Ok(input)
|
||||
}
|
||||
|
||||
/// Runs execution for a single transaction, spawning it in the prewarm threadpool.
|
||||
#[allow(clippy::too_many_arguments)]
|
||||
fn prewarm_transaction(
|
||||
&self,
|
||||
block: N::BlockHeader,
|
||||
tx: N::SignedTx,
|
||||
sender: Address,
|
||||
caches: ProviderCaches,
|
||||
cache_metrics: CachedStateMetrics,
|
||||
state_root_sender: Option<Sender<StateRootMessage>>,
|
||||
cancel_execution: Cancelled,
|
||||
) -> Result<(), InsertBlockErrorKind> {
|
||||
let Some(state_provider) = self.state_provider(block.parent_hash())? else {
|
||||
trace!(target: "engine::tree", parent=%block.parent_hash(), "Could not get state provider for prewarm");
|
||||
return Ok(())
|
||||
};
|
||||
|
||||
// Use the caches to create a new executor
|
||||
let state_provider =
|
||||
CachedStateProvider::new_with_caches(state_provider, caches, cache_metrics);
|
||||
|
||||
// clone and copy info required for execution
|
||||
let evm_config = self.evm_config.clone();
|
||||
|
||||
// spawn task executing the individual tx
|
||||
self.thread_pool.spawn(move || {
|
||||
let state_provider = StateProviderDatabase::new(&state_provider);
|
||||
|
||||
// create a new executor and disable nonce checks in the env
|
||||
let mut evm = evm_config.evm_for_block(state_provider, &block);
|
||||
|
||||
// create the tx env and reset nonce
|
||||
let mut tx_env = evm_config.tx_env(&tx, sender);
|
||||
tx_env.unset_nonce();
|
||||
|
||||
// exit early if execution is done
|
||||
if cancel_execution.is_cancelled() {
|
||||
return
|
||||
}
|
||||
|
||||
let ResultAndState { state, .. } = match evm.transact(tx_env) {
|
||||
Ok(res) => res,
|
||||
Err(err) => {
|
||||
trace!(target: "engine::tree", %err, tx_hash=%tx.tx_hash(), %sender, "Error when executing prewarm transaction");
|
||||
return
|
||||
}
|
||||
};
|
||||
|
||||
// if execution is finished there is no point to sending proof targets
|
||||
if cancel_execution.is_cancelled() {
|
||||
return
|
||||
}
|
||||
|
||||
let Some(state_root_sender) = state_root_sender else {
|
||||
return
|
||||
};
|
||||
|
||||
let mut targets = MultiProofTargets::default();
|
||||
for (addr, account) in state {
|
||||
// if account was not touched, do not fetch for it
|
||||
if !account.is_touched() {
|
||||
continue
|
||||
}
|
||||
|
||||
let mut storage_set = B256Set::default();
|
||||
for (key, slot) in account.storage {
|
||||
// do nothing if unchanged
|
||||
if !slot.is_changed() {
|
||||
continue
|
||||
}
|
||||
|
||||
storage_set.insert(keccak256(B256::new(key.to_be_bytes())));
|
||||
}
|
||||
|
||||
targets.insert(keccak256(addr), storage_set);
|
||||
}
|
||||
|
||||
let _ = state_root_sender.send(StateRootMessage::PrefetchProofs(targets));
|
||||
});
|
||||
|
||||
Ok(())
|
||||
}
|
||||
|
||||
/// Handles an error that occurred while inserting a block.
|
||||
///
|
||||
/// If this is a validation error this will mark the block as invalid.
|
||||
@ -2925,6 +3063,7 @@ mod tests {
|
||||
use reth_ethereum_engine_primitives::{EthEngineTypes, EthereumEngineValidator};
|
||||
use reth_ethereum_primitives::{Block, EthPrimitives};
|
||||
use reth_evm::test_utils::MockExecutorProvider;
|
||||
use reth_evm_ethereum::EthEvmConfig;
|
||||
use reth_primitives_traits::Block as _;
|
||||
use reth_provider::test_utils::MockEthProvider;
|
||||
use reth_trie::{updates::TrieUpdates, HashedPostState};
|
||||
@ -2995,6 +3134,7 @@ mod tests {
|
||||
MockExecutorProvider,
|
||||
EthEngineTypes,
|
||||
EthereumEngineValidator,
|
||||
EthEvmConfig,
|
||||
>,
|
||||
to_tree_tx: Sender<FromEngine<EngineApiRequest<EthEngineTypes, EthPrimitives>, Block>>,
|
||||
from_tree_rx: UnboundedReceiver<EngineApiEvent>,
|
||||
@ -3041,6 +3181,8 @@ mod tests {
|
||||
let (to_payload_service, _payload_command_rx) = unbounded_channel();
|
||||
let payload_builder = PayloadBuilderHandle::new(to_payload_service);
|
||||
|
||||
let evm_config = EthEvmConfig::new(chain_spec.clone());
|
||||
|
||||
let tree = EngineApiTreeHandler::new(
|
||||
provider.clone(),
|
||||
executor_provider.clone(),
|
||||
@ -3054,6 +3196,7 @@ mod tests {
|
||||
payload_builder,
|
||||
TreeConfig::default(),
|
||||
EngineApiKind::Ethereum,
|
||||
evm_config,
|
||||
);
|
||||
|
||||
let block_builder = TestBlockBuilder::default().with_chain_spec((*chain_spec).clone());
|
||||
|
||||
@ -509,6 +509,11 @@ where
|
||||
}
|
||||
}
|
||||
|
||||
/// Returns a [`Sender`] that can be used to send arbitrary [`StateRootMessage`]s to this task.
|
||||
pub fn state_root_message_sender(&self) -> Sender<StateRootMessage> {
|
||||
self.tx.clone()
|
||||
}
|
||||
|
||||
/// Returns a [`StateHookSender`] that can be used to send state updates to this task.
|
||||
pub fn state_hook_sender(&self) -> StateHookSender {
|
||||
StateHookSender::new(self.tx.clone())
|
||||
|
||||
@ -565,6 +565,7 @@ where
|
||||
.with_persistence_threshold(builder.config.engine.persistence_threshold)
|
||||
.with_memory_block_buffer_target(builder.config.engine.memory_block_buffer_target)
|
||||
.with_state_root_task(builder.config.engine.state_root_task_enabled)
|
||||
.with_caching_and_prewarming(builder.config.engine.caching_and_prewarming_enabled)
|
||||
.with_always_compare_trie_updates(
|
||||
builder.config.engine.state_root_task_compare_updates,
|
||||
);
|
||||
|
||||
@ -227,6 +227,7 @@ where
|
||||
Box::pin(consensus_engine_stream),
|
||||
ctx.dev_mining_mode(ctx.components().pool()),
|
||||
LocalPayloadAttributesBuilder::new(ctx.chain_spec()),
|
||||
ctx.components().evm_config().clone(),
|
||||
);
|
||||
|
||||
Either::Left(eth_service)
|
||||
@ -247,6 +248,7 @@ where
|
||||
engine_tree_config,
|
||||
ctx.invalid_block_hook()?,
|
||||
ctx.sync_metrics_tx(),
|
||||
ctx.components().evm_config().clone(),
|
||||
);
|
||||
|
||||
Either::Right(eth_service)
|
||||
|
||||
@ -38,7 +38,9 @@ use reth_provider::{
|
||||
HashedPostStateProvider, ProviderError, StateProofProvider, StateProviderFactory,
|
||||
StateRootProvider,
|
||||
};
|
||||
use reth_revm::{database::StateProviderDatabase, witness::ExecutionWitnessRecord};
|
||||
use reth_revm::{
|
||||
cancelled::Cancelled, database::StateProviderDatabase, witness::ExecutionWitnessRecord,
|
||||
};
|
||||
use reth_transaction_pool::{
|
||||
pool::BestPayloadTransactions, BestTransactionsAttributes, PoolTransaction, TransactionPool,
|
||||
};
|
||||
|
||||
@ -22,7 +22,7 @@ use reth_payload_primitives::{BuiltPayload, PayloadBuilderAttributes, PayloadKin
|
||||
use reth_primitives::{NodePrimitives, SealedHeader};
|
||||
use reth_primitives_traits::proofs;
|
||||
use reth_provider::{BlockReaderIdExt, CanonStateNotification, StateProviderFactory};
|
||||
use reth_revm::cached::CachedReads;
|
||||
use reth_revm::{cached::CachedReads, cancelled::Cancelled};
|
||||
use reth_tasks::TaskSpawner;
|
||||
use reth_transaction_pool::TransactionPool;
|
||||
use revm::{Database, State};
|
||||
@ -31,7 +31,7 @@ use std::{
|
||||
future::Future,
|
||||
ops::Deref,
|
||||
pin::Pin,
|
||||
sync::{atomic::AtomicBool, Arc},
|
||||
sync::Arc,
|
||||
task::{Context, Poll},
|
||||
time::{Duration, SystemTime, UNIX_EPOCH},
|
||||
};
|
||||
@ -681,27 +681,6 @@ impl<P> Future for PendingPayload<P> {
|
||||
}
|
||||
}
|
||||
|
||||
/// A marker that can be used to cancel a job.
|
||||
///
|
||||
/// If dropped, it will set the `cancelled` flag to true.
|
||||
#[derive(Default, Clone, Debug)]
|
||||
pub struct Cancelled(Arc<AtomicBool>);
|
||||
|
||||
// === impl Cancelled ===
|
||||
|
||||
impl Cancelled {
|
||||
/// Returns true if the job was cancelled.
|
||||
pub fn is_cancelled(&self) -> bool {
|
||||
self.0.load(std::sync::atomic::Ordering::Relaxed)
|
||||
}
|
||||
}
|
||||
|
||||
impl Drop for Cancelled {
|
||||
fn drop(&mut self) {
|
||||
self.0.store(true, std::sync::atomic::Ordering::Relaxed);
|
||||
}
|
||||
}
|
||||
|
||||
/// Static config for how to build a payload.
|
||||
#[derive(Clone, Debug)]
|
||||
pub struct PayloadConfig<Attributes> {
|
||||
|
||||
23
crates/revm/src/cancelled.rs
Normal file
23
crates/revm/src/cancelled.rs
Normal file
@ -0,0 +1,23 @@
|
||||
use alloc::sync::Arc;
|
||||
use core::sync::atomic::AtomicBool;
|
||||
|
||||
/// A marker that can be used to cancel execution.
|
||||
///
|
||||
/// If dropped, it will set the `cancelled` flag to true.
|
||||
#[derive(Default, Clone, Debug)]
|
||||
pub struct Cancelled(Arc<AtomicBool>);
|
||||
|
||||
// === impl Cancelled ===
|
||||
|
||||
impl Cancelled {
|
||||
/// Returns true if the job was cancelled.
|
||||
pub fn is_cancelled(&self) -> bool {
|
||||
self.0.load(core::sync::atomic::Ordering::Relaxed)
|
||||
}
|
||||
}
|
||||
|
||||
impl Drop for Cancelled {
|
||||
fn drop(&mut self) {
|
||||
self.0.store(true, core::sync::atomic::Ordering::Relaxed);
|
||||
}
|
||||
}
|
||||
@ -17,6 +17,9 @@ pub mod batch;
|
||||
/// Database adapters for payload building.
|
||||
pub mod cached;
|
||||
|
||||
/// A marker that can be used to cancel execution.
|
||||
pub mod cancelled;
|
||||
|
||||
/// Contains glue code for integrating reth database into revm's [Database].
|
||||
pub mod database;
|
||||
|
||||
|
||||
Reference in New Issue
Block a user