Skip to content

Commit 915b627

Browse files
fix: Revert "feat(engine): improve payload validator tracing spans (#18960)" (#19145)
1 parent e198a38 commit 915b627

File tree

29 files changed

+101
-249
lines changed

29 files changed

+101
-249
lines changed

crates/engine/tree/src/chain.rs

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -71,7 +71,7 @@ where
7171
/// Internal function used to advance the chain.
7272
///
7373
/// Polls the `ChainOrchestrator` for the next event.
74-
#[tracing::instrument(name = "ChainOrchestrator::poll", skip(self, cx))]
74+
#[tracing::instrument(level = "debug", name = "ChainOrchestrator::poll", skip(self, cx))]
7575
fn poll_next_event(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<ChainEvent<T::Event>> {
7676
let this = self.get_mut();
7777

crates/engine/tree/src/tree/cached_state.rs

Lines changed: 2 additions & 16 deletions
Original file line numberDiff line numberDiff line change
@@ -18,7 +18,7 @@ use reth_trie::{
1818
MultiProofTargets, StorageMultiProof, StorageProof, TrieInput,
1919
};
2020
use std::{sync::Arc, time::Duration};
21-
use tracing::{debug_span, instrument, trace};
21+
use tracing::trace;
2222

2323
pub(crate) type Cache<K, V> =
2424
mini_moka::sync::Cache<K, V, alloy_primitives::map::DefaultHashBuilder>;
@@ -354,7 +354,6 @@ impl ExecutionCache {
354354
}
355355

356356
/// Invalidates the storage for all addresses in the set
357-
#[instrument(level = "debug", target = "engine::tree", skip_all, fields(accounts = addresses.len()))]
358357
pub(crate) fn invalidate_storages(&self, addresses: HashSet<&Address>) {
359358
// NOTE: this must collect because the invalidate function should not be called while we
360359
// hold an iter for it
@@ -386,25 +385,12 @@ impl ExecutionCache {
386385
/// ## Error Handling
387386
///
388387
/// Returns an error if the state updates are inconsistent and should be discarded.
389-
#[instrument(level = "debug", target = "engine::tree", skip_all)]
390388
pub(crate) fn insert_state(&self, state_updates: &BundleState) -> Result<(), ()> {
391-
let _enter =
392-
debug_span!(target: "engine::tree", "contracts", len = state_updates.contracts.len())
393-
.entered();
394389
// Insert bytecodes
395390
for (code_hash, bytecode) in &state_updates.contracts {
396391
self.code_cache.insert(*code_hash, Some(Bytecode(bytecode.clone())));
397392
}
398-
drop(_enter);
399-
400-
let _enter = debug_span!(
401-
target: "engine::tree",
402-
"accounts",
403-
accounts = state_updates.state.len(),
404-
storages =
405-
state_updates.state.values().map(|account| account.storage.len()).sum::<usize>()
406-
)
407-
.entered();
393+
408394
let mut invalidated_accounts = HashSet::default();
409395
for (addr, account) in &state_updates.state {
410396
// If the account was not modified, as in not changed and not destroyed, then we have

crates/engine/tree/src/tree/metrics.rs

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -79,7 +79,7 @@ impl EngineApiMetrics {
7979
for tx in transactions {
8080
let tx = tx?;
8181
let span =
82-
debug_span!(target: "engine::tree", "execute tx", tx_hash=?tx.tx().tx_hash());
82+
debug_span!(target: "engine::tree", "execute_tx", tx_hash=?tx.tx().tx_hash());
8383
let _enter = span.enter();
8484
trace!(target: "engine::tree", "Executing transaction");
8585
executor.execute_transaction(tx)?;

crates/engine/tree/src/tree/mod.rs

Lines changed: 4 additions & 11 deletions
Original file line numberDiff line numberDiff line change
@@ -496,12 +496,7 @@ where
496496
///
497497
/// This returns a [`PayloadStatus`] that represents the outcome of a processed new payload and
498498
/// returns an error if an internal error occurred.
499-
#[instrument(
500-
level = "debug",
501-
target = "engine::tree",
502-
skip_all,
503-
fields(block_hash = %payload.block_hash(), block_num = %payload.block_number()),
504-
)]
499+
#[instrument(level = "trace", skip_all, fields(block_hash = %payload.block_hash(), block_num = %payload.block_number(),), target = "engine::tree")]
505500
fn on_new_payload(
506501
&mut self,
507502
payload: T::ExecutionData,
@@ -582,7 +577,6 @@ where
582577
/// - `Valid`: Payload successfully validated and inserted
583578
/// - `Syncing`: Parent missing, payload buffered for later
584579
/// - Error status: Payload is invalid
585-
#[instrument(level = "debug", target = "engine::tree", skip_all)]
586580
fn try_insert_payload(
587581
&mut self,
588582
payload: T::ExecutionData,
@@ -976,7 +970,7 @@ where
976970
/// `engine_forkchoiceUpdated`](https://github.com/ethereum/execution-apis/blob/main/src/engine/paris.md#specification-1).
977971
///
978972
/// Returns an error if an internal error occurred like a database error.
979-
#[instrument(level = "debug", target = "engine::tree", skip_all, fields(head = % state.head_block_hash, safe = % state.safe_block_hash,finalized = % state.finalized_block_hash))]
973+
#[instrument(level = "trace", skip_all, fields(head = % state.head_block_hash, safe = % state.safe_block_hash,finalized = % state.finalized_block_hash), target = "engine::tree")]
980974
fn on_forkchoice_updated(
981975
&mut self,
982976
state: ForkchoiceState,
@@ -1978,7 +1972,7 @@ where
19781972
}
19791973

19801974
/// Attempts to connect any buffered blocks that are connected to the given parent hash.
1981-
#[instrument(level = "debug", target = "engine::tree", skip(self))]
1975+
#[instrument(level = "trace", skip(self), target = "engine::tree")]
19821976
fn try_connect_buffered_blocks(
19831977
&mut self,
19841978
parent: BlockNumHash,
@@ -2287,7 +2281,7 @@ where
22872281
/// Returns an event with the appropriate action to take, such as:
22882282
/// - download more missing blocks
22892283
/// - try to canonicalize the target if the `block` is the tracked target (head) block.
2290-
#[instrument(level = "debug", target = "engine::tree", skip_all, fields(block_hash = %block.hash(), block_num = %block.number()))]
2284+
#[instrument(level = "trace", skip_all, fields(block_hash = %block.hash(), block_num = %block.number(),), target = "engine::tree")]
22912285
fn on_downloaded_block(
22922286
&mut self,
22932287
block: RecoveredBlock<N::Block>,
@@ -2393,7 +2387,6 @@ where
23932387
/// Returns `InsertPayloadOk::Inserted(BlockStatus::Valid)` on successful execution,
23942388
/// `InsertPayloadOk::AlreadySeen` if the block already exists, or
23952389
/// `InsertPayloadOk::Inserted(BlockStatus::Disconnected)` if parent state is missing.
2396-
#[instrument(level = "debug", target = "engine::tree", skip_all, fields(block_id))]
23972390
fn insert_block_or_payload<Input, Err>(
23982391
&mut self,
23992392
block_id: BlockWithParent,

crates/engine/tree/src/tree/payload_processor/mod.rs

Lines changed: 4 additions & 22 deletions
Original file line numberDiff line numberDiff line change
@@ -45,7 +45,7 @@ use std::sync::{
4545
mpsc::{self, channel, Sender},
4646
Arc,
4747
};
48-
use tracing::{debug, debug_span, instrument, warn};
48+
use tracing::{debug, instrument, warn};
4949

5050
mod configured_sparse_trie;
5151
pub mod executor;
@@ -167,12 +167,6 @@ where
167167
/// This returns a handle to await the final state root and to interact with the tasks (e.g.
168168
/// canceling)
169169
#[allow(clippy::type_complexity)]
170-
#[instrument(
171-
level = "debug",
172-
target = "engine::tree::payload_processor",
173-
name = "payload processor",
174-
skip_all
175-
)]
176170
pub fn spawn<P, I: ExecutableTxIterator<Evm>>(
177171
&mut self,
178172
env: ExecutionEnv<Evm>,
@@ -242,9 +236,7 @@ where
242236
);
243237

244238
// spawn multi-proof task
245-
let span = tracing::Span::current();
246239
self.executor.spawn_blocking(move || {
247-
let _enter = span.entered();
248240
multi_proof_task.run();
249241
});
250242

@@ -265,7 +257,6 @@ where
265257
/// Spawns a task that exclusively handles cache prewarming for transaction execution.
266258
///
267259
/// Returns a [`PayloadHandle`] to communicate with the task.
268-
#[instrument(level = "debug", target = "engine::tree::payload_processor", skip_all)]
269260
pub(super) fn spawn_cache_exclusive<P, I: ExecutableTxIterator<Evm>>(
270261
&self,
271262
env: ExecutionEnv<Evm>,
@@ -362,9 +353,7 @@ where
362353
// spawn pre-warm task
363354
{
364355
let to_prewarm_task = to_prewarm_task.clone();
365-
let span = debug_span!(target: "engine::tree::payload_processor", "prewarm task");
366356
self.executor.spawn_blocking(move || {
367-
let _enter = span.entered();
368357
prewarm_task.run(transactions, to_prewarm_task);
369358
});
370359
}
@@ -381,7 +370,7 @@ where
381370
///
382371
/// If the given hash is different then what is recently cached, then this will create a new
383372
/// instance.
384-
#[instrument(level = "debug", target = "engine::caching", skip(self))]
373+
#[instrument(target = "engine::caching", skip(self))]
385374
fn cache_for(&self, parent_hash: B256) -> SavedCache {
386375
if let Some(cache) = self.execution_cache.get_cache_for(parent_hash) {
387376
debug!("reusing execution cache");
@@ -394,7 +383,6 @@ where
394383
}
395384

396385
/// Spawns the [`SparseTrieTask`] for this payload processor.
397-
#[instrument(level = "debug", target = "engine::tree::payload_processor", skip_all)]
398386
fn spawn_sparse_trie_task<BPF>(
399387
&self,
400388
sparse_trie_rx: mpsc::Receiver<SparseTrieUpdate>,
@@ -433,18 +421,13 @@ where
433421
sparse_state_trie,
434422
);
435423

436-
let span = tracing::Span::current();
437424
self.executor.spawn_blocking(move || {
438-
let _enter = span.entered();
439-
440425
let (result, trie) = task.run();
441426
// Send state root computation result
442427
let _ = state_root_tx.send(result);
443428

444-
// Clear the SparseStateTrie and replace it back into the mutex _after_ sending
445-
// results to the next step, so that time spent clearing doesn't block the step after
446-
// this one.
447-
let _enter = debug_span!(target: "engine::tree::payload_processor", "clear").entered();
429+
// Clear the SparseStateTrie and replace it back into the mutex _after_ sending results
430+
// to the next step, so that time spent clearing doesn't block the step after this one.
448431
cleared_sparse_trie.lock().replace(ClearedSparseStateTrie::from_state_trie(trie));
449432
});
450433
}
@@ -469,7 +452,6 @@ impl<Tx, Err> PayloadHandle<Tx, Err> {
469452
/// # Panics
470453
///
471454
/// If payload processing was started without background tasks.
472-
#[instrument(level = "debug", target = "engine::tree::payload_processor", skip_all)]
473455
pub fn state_root(&mut self) -> Result<StateRootComputeOutcome, ParallelStateRootError> {
474456
self.state_root
475457
.take()

crates/engine/tree/src/tree/payload_processor/multiproof.rs

Lines changed: 1 addition & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -32,7 +32,7 @@ use std::{
3232
},
3333
time::{Duration, Instant},
3434
};
35-
use tracing::{debug, error, instrument, trace};
35+
use tracing::{debug, error, trace};
3636

3737
/// A trie update that can be applied to sparse trie alongside the proofs for touched parts of the
3838
/// state.
@@ -718,7 +718,6 @@ impl MultiProofTask {
718718
/// Handles request for proof prefetch.
719719
///
720720
/// Returns a number of proofs that were spawned.
721-
#[instrument(level = "debug", target = "engine::tree::payload_processor::multiproof", skip_all, fields(accounts = targets.len()))]
722721
fn on_prefetch_proof(&mut self, targets: MultiProofTargets) -> u64 {
723722
let proof_targets = self.get_prefetch_proof_targets(targets);
724723
self.fetched_proof_targets.extend_ref(&proof_targets);
@@ -845,7 +844,6 @@ impl MultiProofTask {
845844
/// Handles state updates.
846845
///
847846
/// Returns a number of proofs that were spawned.
848-
#[instrument(level = "debug", target = "engine::tree::payload_processor::multiproof", skip(self, update), fields(accounts = update.len()))]
849847
fn on_state_update(&mut self, source: StateChangeSource, update: EvmState) -> u64 {
850848
let hashed_state_update = evm_state_to_hashed_post_state(update);
851849

@@ -975,7 +973,6 @@ impl MultiProofTask {
975973
/// currently being calculated, or if there are any pending proofs in the proof sequencer
976974
/// left to be revealed by checking the pending tasks.
977975
/// 6. This task exits after all pending proofs are processed.
978-
#[instrument(level = "debug", target = "engine::tree::payload_processor::multiproof", skip_all)]
979976
pub(crate) fn run(mut self) {
980977
// TODO convert those into fields
981978
let mut prefetch_proofs_requested = 0;

crates/engine/tree/src/tree/payload_processor/prewarm.rs

Lines changed: 5 additions & 35 deletions
Original file line numberDiff line numberDiff line change
@@ -39,7 +39,7 @@ use std::{
3939
},
4040
time::Instant,
4141
};
42-
use tracing::{debug, debug_span, instrument, trace, warn};
42+
use tracing::{debug, trace, warn};
4343

4444
/// A wrapper for transactions that includes their index in the block.
4545
#[derive(Clone)]
@@ -139,11 +139,8 @@ where
139139
let ctx = self.ctx.clone();
140140
let max_concurrency = self.max_concurrency;
141141
let transaction_count_hint = self.transaction_count_hint;
142-
let span = tracing::Span::current();
143142

144143
self.executor.spawn_blocking(move || {
145-
let _enter = debug_span!(target: "engine::tree::payload_processor::prewarm", parent: span, "spawn_all").entered();
146-
147144
let (done_tx, done_rx) = mpsc::channel();
148145
let mut executing = 0usize;
149146

@@ -160,8 +157,8 @@ where
160157
};
161158

162159
// Only spawn initial workers as needed
163-
for i in 0..workers_needed {
164-
handles.push(ctx.spawn_worker(i, &executor, actions_tx.clone(), done_tx.clone()));
160+
for _ in 0..workers_needed {
161+
handles.push(ctx.spawn_worker(&executor, actions_tx.clone(), done_tx.clone()));
165162
}
166163

167164
let mut tx_index = 0usize;
@@ -251,7 +248,6 @@ where
251248
/// the new, warmed cache to be inserted.
252249
///
253250
/// This method is called from `run()` only after all execution tasks are complete.
254-
#[instrument(level = "debug", target = "engine::tree::payload_processor::prewarm", skip_all)]
255251
fn save_cache(self, state: BundleState) {
256252
let start = Instant::now();
257253

@@ -288,12 +284,6 @@ where
288284
///
289285
/// This will execute the transactions until all transactions have been processed or the task
290286
/// was cancelled.
291-
#[instrument(
292-
level = "debug",
293-
target = "engine::tree::payload_processor::prewarm",
294-
name = "prewarm",
295-
skip_all
296-
)]
297287
pub(super) fn run(
298288
self,
299289
pending: mpsc::Receiver<impl ExecutableTxFor<Evm> + Clone + Send + 'static>,
@@ -374,7 +364,6 @@ where
374364
{
375365
/// Splits this context into an evm, an evm config, metrics, and the atomic bool for terminating
376366
/// execution.
377-
#[instrument(level = "debug", target = "engine::tree::payload_processor::prewarm", skip_all)]
378367
fn evm_for_ctx(self) -> Option<(EvmFor<Evm, impl Database>, PrewarmMetrics, Arc<AtomicBool>)> {
379368
let Self {
380369
env,
@@ -391,7 +380,7 @@ where
391380
Ok(provider) => provider,
392381
Err(err) => {
393382
trace!(
394-
target: "engine::tree::payload_processor::prewarm",
383+
target: "engine::tree",
395384
%err,
396385
"Failed to build state provider in prewarm thread"
397386
);
@@ -440,7 +429,6 @@ where
440429
///
441430
/// Note: There are no ordering guarantees; this does not reflect the state produced by
442431
/// sequential execution.
443-
#[instrument(level = "debug", target = "engine::tree::payload_processor::prewarm", skip_all)]
444432
fn transact_batch<Tx>(
445433
self,
446434
txs: mpsc::Receiver<IndexedTransaction<Tx>>,
@@ -451,15 +439,7 @@ where
451439
{
452440
let Some((mut evm, metrics, terminate_execution)) = self.evm_for_ctx() else { return };
453441

454-
while let Ok(IndexedTransaction { index, tx }) = {
455-
let _enter = debug_span!(target: "engine::tree::payload_processor::prewarm", "recv tx")
456-
.entered();
457-
txs.recv()
458-
} {
459-
let _enter =
460-
debug_span!(target: "engine::tree::payload_processor::prewarm", "prewarm tx", index, tx_hash=%tx.tx().tx_hash())
461-
.entered();
462-
442+
while let Ok(IndexedTransaction { index, tx }) = txs.recv() {
463443
// If the task was cancelled, stop execution, send an empty result to notify the task,
464444
// and exit.
465445
if terminate_execution.load(Ordering::Relaxed) {
@@ -487,18 +467,12 @@ where
487467
};
488468
metrics.execution_duration.record(start.elapsed());
489469

490-
drop(_enter);
491-
492470
// Only send outcome for transactions after the first txn
493471
// as the main execution will be just as fast
494472
if index > 0 {
495-
let _enter =
496-
debug_span!(target: "engine::tree::payload_processor::prewarm", "prewarm outcome", index, tx_hash=%tx.tx().tx_hash())
497-
.entered();
498473
let (targets, storage_targets) = multiproof_targets_from_state(res.state);
499474
metrics.prefetch_storage_targets.record(storage_targets as f64);
500475
let _ = sender.send(PrewarmTaskEvent::Outcome { proof_targets: Some(targets) });
501-
drop(_enter);
502476
}
503477

504478
metrics.total_runtime.record(start.elapsed());
@@ -511,7 +485,6 @@ where
511485
/// Spawns a worker task for transaction execution and returns its sender channel.
512486
fn spawn_worker<Tx>(
513487
&self,
514-
idx: usize,
515488
executor: &WorkloadExecutor,
516489
actions_tx: Sender<PrewarmTaskEvent>,
517490
done_tx: Sender<()>,
@@ -521,11 +494,8 @@ where
521494
{
522495
let (tx, rx) = mpsc::channel();
523496
let ctx = self.clone();
524-
let span =
525-
debug_span!(target: "engine::tree::payload_processor::prewarm", "prewarm worker", idx);
526497

527498
executor.spawn_blocking(move || {
528-
let _enter = span.entered();
529499
ctx.transact_batch(rx, actions_tx, done_tx);
530500
});
531501

0 commit comments

Comments
 (0)