mirror of
https://github.com/hl-archive-node/nanoreth.git
synced 2025-12-06 10:59:55 +00:00
feat: add basic payload generator (#2161)
This commit is contained in:
21
Cargo.lock
generated
21
Cargo.lock
generated
@ -4489,6 +4489,25 @@ dependencies = [
|
|||||||
"tracing",
|
"tracing",
|
||||||
]
|
]
|
||||||
|
|
||||||
|
[[package]]
|
||||||
|
name = "reth-basic-payload-builder"
|
||||||
|
version = "0.1.0"
|
||||||
|
dependencies = [
|
||||||
|
"futures-core",
|
||||||
|
"futures-util",
|
||||||
|
"reth-consensus-common",
|
||||||
|
"reth-miner",
|
||||||
|
"reth-primitives",
|
||||||
|
"reth-provider",
|
||||||
|
"reth-revm",
|
||||||
|
"reth-rlp",
|
||||||
|
"reth-tasks",
|
||||||
|
"reth-transaction-pool",
|
||||||
|
"revm",
|
||||||
|
"tokio",
|
||||||
|
"tracing",
|
||||||
|
]
|
||||||
|
|
||||||
[[package]]
|
[[package]]
|
||||||
name = "reth-beacon-consensus"
|
name = "reth-beacon-consensus"
|
||||||
version = "0.1.0"
|
version = "0.1.0"
|
||||||
@ -4840,9 +4859,11 @@ dependencies = [
|
|||||||
"futures-core",
|
"futures-core",
|
||||||
"futures-util",
|
"futures-util",
|
||||||
"parking_lot 0.12.1",
|
"parking_lot 0.12.1",
|
||||||
|
"reth-interfaces",
|
||||||
"reth-primitives",
|
"reth-primitives",
|
||||||
"reth-rlp",
|
"reth-rlp",
|
||||||
"reth-rpc-types",
|
"reth-rpc-types",
|
||||||
|
"revm-primitives",
|
||||||
"sha2 0.10.6",
|
"sha2 0.10.6",
|
||||||
"thiserror",
|
"thiserror",
|
||||||
"tokio",
|
"tokio",
|
||||||
|
|||||||
@ -18,6 +18,7 @@ members = [
|
|||||||
"crates/net/network-api",
|
"crates/net/network-api",
|
||||||
"crates/net/network",
|
"crates/net/network",
|
||||||
"crates/net/downloaders",
|
"crates/net/downloaders",
|
||||||
|
"crates/payload/basic",
|
||||||
"crates/primitives",
|
"crates/primitives",
|
||||||
"crates/revm",
|
"crates/revm",
|
||||||
"crates/revm/revm-primitives",
|
"crates/revm/revm-primitives",
|
||||||
|
|||||||
@ -12,6 +12,10 @@ description = "Block production"
|
|||||||
reth-primitives = { path = "../primitives" }
|
reth-primitives = { path = "../primitives" }
|
||||||
reth-rpc-types = { path = "../rpc/rpc-types" }
|
reth-rpc-types = { path = "../rpc/rpc-types" }
|
||||||
reth-rlp = { path = "../rlp" }
|
reth-rlp = { path = "../rlp" }
|
||||||
|
reth-interfaces = { path = "../interfaces" }
|
||||||
|
|
||||||
|
## ethereum
|
||||||
|
revm-primitives = "1"
|
||||||
|
|
||||||
## async
|
## async
|
||||||
tokio = { version = "1", features = ["sync"] }
|
tokio = { version = "1", features = ["sync"] }
|
||||||
|
|||||||
@ -1,13 +1,32 @@
|
|||||||
//! Error types emitted by types or implementations of this crate.
|
//! Error types emitted by types or implementations of this crate.
|
||||||
|
|
||||||
|
use reth_primitives::H256;
|
||||||
|
use revm_primitives::EVMError;
|
||||||
use tokio::sync::oneshot;
|
use tokio::sync::oneshot;
|
||||||
|
|
||||||
/// Possible error variants during payload building.
|
/// Possible error variants during payload building.
|
||||||
#[derive(Debug, thiserror::Error)]
|
#[derive(Debug, thiserror::Error)]
|
||||||
pub enum PayloadBuilderError {
|
pub enum PayloadBuilderError {
|
||||||
/// A oneshot channels has been closed.
|
/// Thrown whe the parent block is missing.
|
||||||
#[error("Sender has been dropped")]
|
#[error("missing parent block {0:?}")]
|
||||||
|
MissingParentBlock(H256),
|
||||||
|
/// An oneshot channels has been closed.
|
||||||
|
#[error("sender has been dropped")]
|
||||||
ChannelClosed,
|
ChannelClosed,
|
||||||
|
/// Other internal error
|
||||||
|
#[error(transparent)]
|
||||||
|
Internal(#[from] reth_interfaces::Error),
|
||||||
|
|
||||||
|
// TODO move to standalone error type specific to job
|
||||||
|
/// Thrown if a running build job has been cancelled.
|
||||||
|
#[error("build job cancelled during execution")]
|
||||||
|
BuildJobCancelled,
|
||||||
|
/// Unrecoverable error during evm execution.
|
||||||
|
#[error("evm execution error: {0:?}")]
|
||||||
|
EvmExecutionError(EVMError<reth_interfaces::Error>),
|
||||||
|
/// Thrown if the payload requests withdrawals before Shanghai activation.
|
||||||
|
#[error("withdrawals set before Shanghai activation")]
|
||||||
|
WithdrawalsBeforeShanghai,
|
||||||
}
|
}
|
||||||
|
|
||||||
impl From<oneshot::error::RecvError> for PayloadBuilderError {
|
impl From<oneshot::error::RecvError> for PayloadBuilderError {
|
||||||
|
|||||||
@ -1,6 +1,6 @@
|
|||||||
//! Contains types required for building a payload.
|
//! Contains types required for building a payload.
|
||||||
|
|
||||||
use reth_primitives::{Address, Block, SealedBlock, Withdrawal, H256, U256};
|
use reth_primitives::{Address, SealedBlock, Withdrawal, H256, U256};
|
||||||
use reth_rlp::Encodable;
|
use reth_rlp::Encodable;
|
||||||
use reth_rpc_types::engine::{PayloadAttributes, PayloadId};
|
use reth_rpc_types::engine::{PayloadAttributes, PayloadId};
|
||||||
|
|
||||||
@ -23,8 +23,8 @@ pub struct BuiltPayload {
|
|||||||
|
|
||||||
impl BuiltPayload {
|
impl BuiltPayload {
|
||||||
/// Initializes the payload with the given initial block.
|
/// Initializes the payload with the given initial block.
|
||||||
pub(crate) fn new(id: PayloadId, block: Block, fees: U256) -> Self {
|
pub fn new(id: PayloadId, block: SealedBlock, fees: U256) -> Self {
|
||||||
Self { id, block: block.seal_slow(), fees }
|
Self { id, block, fees }
|
||||||
}
|
}
|
||||||
|
|
||||||
/// Returns the identifier of the payload.
|
/// Returns the identifier of the payload.
|
||||||
@ -46,25 +46,30 @@ impl BuiltPayload {
|
|||||||
/// Container type for all components required to build a payload.
|
/// Container type for all components required to build a payload.
|
||||||
#[derive(Debug, Clone, PartialEq, Eq)]
|
#[derive(Debug, Clone, PartialEq, Eq)]
|
||||||
pub struct PayloadBuilderAttributes {
|
pub struct PayloadBuilderAttributes {
|
||||||
// TODO include id here
|
/// Id of the payload
|
||||||
|
pub id: PayloadId,
|
||||||
/// Parent block to build the payload on top
|
/// Parent block to build the payload on top
|
||||||
pub(crate) parent: H256,
|
pub parent: H256,
|
||||||
/// Timestamp for the generated payload
|
/// Timestamp for the generated payload
|
||||||
pub(crate) timestamp: u64,
|
pub timestamp: u64,
|
||||||
/// Address of the recipient for collecting transaction fee
|
/// Address of the recipient for collecting transaction fee
|
||||||
pub(crate) suggested_fee_recipient: Address,
|
pub suggested_fee_recipient: Address,
|
||||||
/// Randomness value for the generated payload
|
/// Randomness value for the generated payload
|
||||||
pub(crate) prev_randao: H256,
|
pub prev_randao: H256,
|
||||||
/// Withdrawals for the generated payload
|
/// Withdrawals for the generated payload
|
||||||
pub(crate) withdrawals: Vec<Withdrawal>,
|
pub withdrawals: Vec<Withdrawal>,
|
||||||
}
|
}
|
||||||
|
|
||||||
// === impl PayloadBuilderAttributes ===
|
// === impl PayloadBuilderAttributes ===
|
||||||
|
|
||||||
impl PayloadBuilderAttributes {
|
impl PayloadBuilderAttributes {
|
||||||
/// Creates a new payload builder for the given parent block and the attributes
|
/// Creates a new payload builder for the given parent block and the attributes.
|
||||||
|
///
|
||||||
|
/// Derives the unique [PayloadId] for the given parent and attributes
|
||||||
pub fn new(parent: H256, attributes: PayloadAttributes) -> Self {
|
pub fn new(parent: H256, attributes: PayloadAttributes) -> Self {
|
||||||
|
let id = payload_id(&parent, &attributes);
|
||||||
Self {
|
Self {
|
||||||
|
id,
|
||||||
parent,
|
parent,
|
||||||
timestamp: attributes.timestamp.as_u64(),
|
timestamp: attributes.timestamp.as_u64(),
|
||||||
suggested_fee_recipient: attributes.suggested_fee_recipient,
|
suggested_fee_recipient: attributes.suggested_fee_recipient,
|
||||||
@ -73,20 +78,27 @@ impl PayloadBuilderAttributes {
|
|||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
/// Generates the payload id for the configured payload
|
/// Returns the identifier of the payload.
|
||||||
///
|
pub fn payload_id(&self) -> PayloadId {
|
||||||
/// Returns an 8-byte identifier by hashing the payload components.
|
self.id
|
||||||
pub(crate) fn payload_id(&self) -> PayloadId {
|
|
||||||
use sha2::Digest;
|
|
||||||
let mut hasher = sha2::Sha256::new();
|
|
||||||
hasher.update(self.parent.as_bytes());
|
|
||||||
hasher.update(&self.timestamp.to_be_bytes()[..]);
|
|
||||||
hasher.update(self.prev_randao.as_bytes());
|
|
||||||
hasher.update(self.suggested_fee_recipient.as_bytes());
|
|
||||||
let mut buf = Vec::new();
|
|
||||||
self.withdrawals.encode(&mut buf);
|
|
||||||
hasher.update(buf);
|
|
||||||
let out = hasher.finalize();
|
|
||||||
PayloadId::new(out.as_slice()[..8].try_into().expect("sufficient length"))
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/// Generates the payload id for the configured payload
|
||||||
|
///
|
||||||
|
/// Returns an 8-byte identifier by hashing the payload components.
|
||||||
|
pub(crate) fn payload_id(parent: &H256, attributes: &PayloadAttributes) -> PayloadId {
|
||||||
|
use sha2::Digest;
|
||||||
|
let mut hasher = sha2::Sha256::new();
|
||||||
|
hasher.update(parent.as_bytes());
|
||||||
|
hasher.update(&attributes.timestamp.as_u64().to_be_bytes()[..]);
|
||||||
|
hasher.update(attributes.prev_randao.as_bytes());
|
||||||
|
hasher.update(attributes.suggested_fee_recipient.as_bytes());
|
||||||
|
if let Some(withdrawals) = &attributes.withdrawals {
|
||||||
|
let mut buf = Vec::new();
|
||||||
|
withdrawals.encode(&mut buf);
|
||||||
|
hasher.update(buf);
|
||||||
|
}
|
||||||
|
let out = hasher.finalize();
|
||||||
|
PayloadId::new(out.as_slice()[..8].try_into().expect("sufficient length"))
|
||||||
|
}
|
||||||
|
|||||||
@ -3,7 +3,10 @@
|
|||||||
//! The payload builder is responsible for building payloads.
|
//! The payload builder is responsible for building payloads.
|
||||||
//! Once a new payload is created, it is continuously updated.
|
//! Once a new payload is created, it is continuously updated.
|
||||||
|
|
||||||
use crate::{traits::PayloadJobGenerator, BuiltPayload, PayloadBuilderAttributes, PayloadJob};
|
use crate::{
|
||||||
|
error::PayloadBuilderError, traits::PayloadJobGenerator, BuiltPayload,
|
||||||
|
PayloadBuilderAttributes, PayloadJob,
|
||||||
|
};
|
||||||
use futures_util::stream::{StreamExt, TryStreamExt};
|
use futures_util::stream::{StreamExt, TryStreamExt};
|
||||||
use reth_rpc_types::engine::PayloadId;
|
use reth_rpc_types::engine::PayloadId;
|
||||||
use std::{
|
use std::{
|
||||||
@ -58,10 +61,10 @@ impl PayloadBuilderHandle {
|
|||||||
pub async fn new_payload(
|
pub async fn new_payload(
|
||||||
&self,
|
&self,
|
||||||
attr: PayloadBuilderAttributes,
|
attr: PayloadBuilderAttributes,
|
||||||
) -> Result<PayloadId, oneshot::error::RecvError> {
|
) -> Result<PayloadId, PayloadBuilderError> {
|
||||||
let (tx, rx) = oneshot::channel();
|
let (tx, rx) = oneshot::channel();
|
||||||
let _ = self.to_service.send(PayloadServiceCommand::BuildNewPayload(attr, tx));
|
let _ = self.to_service.send(PayloadServiceCommand::BuildNewPayload(attr, tx));
|
||||||
rx.await
|
rx.await?
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@ -168,15 +171,24 @@ where
|
|||||||
match cmd {
|
match cmd {
|
||||||
PayloadServiceCommand::BuildNewPayload(attr, tx) => {
|
PayloadServiceCommand::BuildNewPayload(attr, tx) => {
|
||||||
let id = attr.payload_id();
|
let id = attr.payload_id();
|
||||||
|
let mut res = Ok(id);
|
||||||
|
|
||||||
if !this.contains_payload(id) {
|
if !this.contains_payload(id) {
|
||||||
// no job for this payload yet, create one
|
// no job for this payload yet, create one
|
||||||
new_job = true;
|
match this.generator.new_payload_job(attr) {
|
||||||
let job = this.generator.new_payload_job(attr);
|
Ok(job) => {
|
||||||
this.payload_jobs.push((job, id));
|
new_job = true;
|
||||||
|
this.payload_jobs.push((job, id));
|
||||||
|
}
|
||||||
|
Err(err) => {
|
||||||
|
warn!(?err, %id, "failed to create payload job");
|
||||||
|
res = Err(err);
|
||||||
|
}
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
// return the id of the payload
|
// return the id of the payload
|
||||||
let _ = tx.send(id);
|
let _ = tx.send(res);
|
||||||
}
|
}
|
||||||
PayloadServiceCommand::GetPayload(id, tx) => {
|
PayloadServiceCommand::GetPayload(id, tx) => {
|
||||||
let _ = tx.send(this.get_payload(id));
|
let _ = tx.send(this.get_payload(id));
|
||||||
@ -195,7 +207,10 @@ where
|
|||||||
#[derive(Debug)]
|
#[derive(Debug)]
|
||||||
enum PayloadServiceCommand {
|
enum PayloadServiceCommand {
|
||||||
/// Start building a new payload.
|
/// Start building a new payload.
|
||||||
BuildNewPayload(PayloadBuilderAttributes, oneshot::Sender<PayloadId>),
|
BuildNewPayload(
|
||||||
|
PayloadBuilderAttributes,
|
||||||
|
oneshot::Sender<Result<PayloadId, PayloadBuilderError>>,
|
||||||
|
),
|
||||||
/// Get the current payload.
|
/// Get the current payload.
|
||||||
GetPayload(PayloadId, oneshot::Sender<Option<Arc<BuiltPayload>>>),
|
GetPayload(PayloadId, oneshot::Sender<Option<Arc<BuiltPayload>>>),
|
||||||
}
|
}
|
||||||
|
|||||||
@ -33,5 +33,8 @@ pub trait PayloadJobGenerator: Send + Sync {
|
|||||||
///
|
///
|
||||||
/// Note: this is expected to build a new (empty) payload without transactions, so it can be
|
/// Note: this is expected to build a new (empty) payload without transactions, so it can be
|
||||||
/// returned directly. when asked for
|
/// returned directly. when asked for
|
||||||
fn new_payload_job(&self, attr: PayloadBuilderAttributes) -> Self::Job;
|
fn new_payload_job(
|
||||||
|
&self,
|
||||||
|
attr: PayloadBuilderAttributes,
|
||||||
|
) -> Result<Self::Job, PayloadBuilderError>;
|
||||||
}
|
}
|
||||||
|
|||||||
30
crates/payload/basic/Cargo.toml
Normal file
30
crates/payload/basic/Cargo.toml
Normal file
@ -0,0 +1,30 @@
|
|||||||
|
[package]
|
||||||
|
name = "reth-basic-payload-builder"
|
||||||
|
version = "0.1.0"
|
||||||
|
edition = "2021"
|
||||||
|
license = "MIT OR Apache-2.0"
|
||||||
|
repository = "https://github.com/paradigmxyz/reth"
|
||||||
|
readme = "README.md"
|
||||||
|
description = "A basic payload builder for reth that uses the txpool API to build payloads."
|
||||||
|
|
||||||
|
[dependencies]
|
||||||
|
## reth
|
||||||
|
reth-primitives = { path = "../../primitives" }
|
||||||
|
reth-consensus-common = { path = "../../consensus/common" }
|
||||||
|
reth-revm = { path = "../../revm" }
|
||||||
|
reth-transaction-pool = { path = "../../transaction-pool" }
|
||||||
|
reth-rlp = { path = "../../rlp" }
|
||||||
|
reth-provider = { path = "../../storage/provider" }
|
||||||
|
reth-miner = { path = "../../miner" }
|
||||||
|
reth-tasks = { path = "../../tasks" }
|
||||||
|
|
||||||
|
## ethereum
|
||||||
|
revm = { version = "3" }
|
||||||
|
|
||||||
|
## async
|
||||||
|
tokio = { version = "1", features = ["sync", "time"] }
|
||||||
|
futures-core = "0.3"
|
||||||
|
futures-util = "0.3"
|
||||||
|
|
||||||
|
## misc
|
||||||
|
tracing = "0.1"
|
||||||
561
crates/payload/basic/src/lib.rs
Normal file
561
crates/payload/basic/src/lib.rs
Normal file
@ -0,0 +1,561 @@
|
|||||||
|
#![warn(missing_docs, unreachable_pub)]
|
||||||
|
#![deny(unused_must_use, rust_2018_idioms)]
|
||||||
|
#![doc(test(
|
||||||
|
no_crate_inject,
|
||||||
|
attr(deny(warnings, rust_2018_idioms), allow(dead_code, unused_variables))
|
||||||
|
))]
|
||||||
|
// TODO rm later
|
||||||
|
#![allow(unused)]
|
||||||
|
|
||||||
|
//! reth basic payload job generator
|
||||||
|
|
||||||
|
use futures_core::{ready, Stream};
|
||||||
|
use futures_util::FutureExt;
|
||||||
|
use reth_consensus_common::validation::calculate_next_block_base_fee;
|
||||||
|
use reth_miner::{
|
||||||
|
error::PayloadBuilderError, BuiltPayload, PayloadBuilderAttributes, PayloadJob,
|
||||||
|
PayloadJobGenerator,
|
||||||
|
};
|
||||||
|
use reth_primitives::{
|
||||||
|
bloom::logs_bloom, bytes::Bytes, proofs, Block, ChainSpec, Hardfork, Head, Header,
|
||||||
|
IntoRecoveredTransaction, Receipt, SealedBlock, EMPTY_OMMER_ROOT, U256,
|
||||||
|
};
|
||||||
|
use reth_provider::{BlockProvider, EvmEnvProvider, PostState, StateProviderFactory};
|
||||||
|
use reth_revm::{
|
||||||
|
config::{revm_spec, revm_spec_by_timestamp_after_merge},
|
||||||
|
database::{State, SubState},
|
||||||
|
env::tx_env_with_recovered,
|
||||||
|
executor::{
|
||||||
|
commit_state_changes, increment_account_balance, post_block_withdrawals_balance_increments,
|
||||||
|
},
|
||||||
|
into_reth_log,
|
||||||
|
};
|
||||||
|
use reth_tasks::TaskSpawner;
|
||||||
|
use reth_transaction_pool::TransactionPool;
|
||||||
|
use revm::primitives::{BlockEnv, CfgEnv, Env, ResultAndState, SpecId};
|
||||||
|
use std::{
|
||||||
|
future::Future,
|
||||||
|
pin::Pin,
|
||||||
|
sync::{atomic::AtomicBool, Arc},
|
||||||
|
task::{Context, Poll},
|
||||||
|
time::Duration,
|
||||||
|
};
|
||||||
|
use tokio::{
|
||||||
|
sync::{oneshot, Semaphore},
|
||||||
|
time::{Interval, Sleep},
|
||||||
|
};
|
||||||
|
use tracing::trace;
|
||||||
|
|
||||||
|
// TODO move to common since commonly used
|
||||||
|
|
||||||
|
/// Settings for how to generate a block
|
||||||
|
#[derive(Debug, Clone)]
|
||||||
|
pub struct BlockConfig {
|
||||||
|
/// Data to include in the block's extra data field.
|
||||||
|
extradata: Bytes,
|
||||||
|
/// Target gas ceiling for mined blocks, defaults to 30_000_000 gas.
|
||||||
|
max_gas_limit: u64,
|
||||||
|
}
|
||||||
|
|
||||||
|
/// The [PayloadJobGenerator] that creates [BasicPayloadJob]s.
|
||||||
|
pub struct BasicPayloadJobGenerator<Client, Pool, Tasks> {
|
||||||
|
/// The client that can interact with the chain.
|
||||||
|
client: Client,
|
||||||
|
/// txpool
|
||||||
|
pool: Pool,
|
||||||
|
/// How to spawn building tasks
|
||||||
|
executor: Tasks,
|
||||||
|
/// The configuration for the job generator.
|
||||||
|
config: BasicPayloadJobGeneratorConfig,
|
||||||
|
/// The configuration for how to create a block.
|
||||||
|
block_config: BlockConfig,
|
||||||
|
/// Restricts how many generator tasks can be executed at once.
|
||||||
|
payload_task_guard: PayloadTaskGuard,
|
||||||
|
/// The chain spec.
|
||||||
|
chain_spec: Arc<ChainSpec>,
|
||||||
|
}
|
||||||
|
|
||||||
|
// === impl BasicPayloadJobGenerator ===
|
||||||
|
|
||||||
|
impl<Client, Pool, Tasks> BasicPayloadJobGenerator<Client, Pool, Tasks> {
|
||||||
|
/// Creates a new [BasicPayloadJobGenerator] with the given config.
|
||||||
|
pub fn new(
|
||||||
|
client: Client,
|
||||||
|
pool: Pool,
|
||||||
|
executor: Tasks,
|
||||||
|
config: BasicPayloadJobGeneratorConfig,
|
||||||
|
block_config: BlockConfig,
|
||||||
|
chain_spec: Arc<ChainSpec>,
|
||||||
|
) -> Self {
|
||||||
|
Self {
|
||||||
|
client,
|
||||||
|
pool,
|
||||||
|
executor,
|
||||||
|
payload_task_guard: PayloadTaskGuard::new(config.max_payload_tasks),
|
||||||
|
config,
|
||||||
|
block_config,
|
||||||
|
chain_spec,
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
// === impl BasicPayloadJobGenerator ===
|
||||||
|
|
||||||
|
impl<Client, Pool, Tasks> BasicPayloadJobGenerator<Client, Pool, Tasks> {}
|
||||||
|
|
||||||
|
impl<Client, Pool, Tasks> PayloadJobGenerator for BasicPayloadJobGenerator<Client, Pool, Tasks>
|
||||||
|
where
|
||||||
|
Client: StateProviderFactory + BlockProvider + Clone + Unpin + 'static,
|
||||||
|
Pool: TransactionPool + Unpin + 'static,
|
||||||
|
Tasks: TaskSpawner + Clone + Unpin + 'static,
|
||||||
|
{
|
||||||
|
type Job = BasicPayloadJob<Client, Pool, Tasks>;
|
||||||
|
|
||||||
|
fn new_payload_job(
|
||||||
|
&self,
|
||||||
|
attributes: PayloadBuilderAttributes,
|
||||||
|
) -> Result<Self::Job, PayloadBuilderError> {
|
||||||
|
// TODO this needs to access the _pending_ state of the parent block hash
|
||||||
|
let parent_block = self
|
||||||
|
.client
|
||||||
|
.block_by_hash(attributes.parent)?
|
||||||
|
.ok_or_else(|| PayloadBuilderError::MissingParentBlock(attributes.parent))?;
|
||||||
|
|
||||||
|
// configure evm env based on parent block
|
||||||
|
let initialized_cfg = CfgEnv {
|
||||||
|
chain_id: U256::from(self.chain_spec.chain().id()),
|
||||||
|
// ensure we're not missing any timestamp based hardforks
|
||||||
|
spec_id: revm_spec_by_timestamp_after_merge(&self.chain_spec, attributes.timestamp),
|
||||||
|
..Default::default()
|
||||||
|
};
|
||||||
|
|
||||||
|
let initialized_block_env = BlockEnv {
|
||||||
|
number: U256::from(parent_block.number + 1),
|
||||||
|
coinbase: attributes.suggested_fee_recipient,
|
||||||
|
timestamp: U256::from(attributes.timestamp),
|
||||||
|
difficulty: U256::ZERO,
|
||||||
|
prevrandao: Some(attributes.prev_randao),
|
||||||
|
gas_limit: U256::from(parent_block.gas_limit),
|
||||||
|
// calculate basefee based on parent block's gas usage
|
||||||
|
basefee: U256::from(calculate_next_block_base_fee(
|
||||||
|
parent_block.gas_used,
|
||||||
|
parent_block.gas_limit,
|
||||||
|
parent_block.base_fee_per_gas.unwrap_or_default(),
|
||||||
|
)),
|
||||||
|
};
|
||||||
|
|
||||||
|
let config = PayloadConfig {
|
||||||
|
initialized_block_env,
|
||||||
|
initialized_cfg,
|
||||||
|
parent_block: Arc::new(parent_block),
|
||||||
|
extra_data: self.block_config.extradata.clone(),
|
||||||
|
attributes,
|
||||||
|
chain_spec: Arc::clone(&self.chain_spec),
|
||||||
|
};
|
||||||
|
|
||||||
|
// create empty
|
||||||
|
|
||||||
|
let until = tokio::time::Instant::now() + self.config.deadline;
|
||||||
|
let deadline = Box::pin(tokio::time::sleep_until(until));
|
||||||
|
|
||||||
|
Ok(BasicPayloadJob {
|
||||||
|
config,
|
||||||
|
client: self.client.clone(),
|
||||||
|
pool: self.pool.clone(),
|
||||||
|
executor: self.executor.clone(),
|
||||||
|
deadline,
|
||||||
|
interval: tokio::time::interval(self.config.interval),
|
||||||
|
best_payload: None,
|
||||||
|
pending_block: None,
|
||||||
|
payload_task_guard: self.payload_task_guard.clone(),
|
||||||
|
})
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
/// Restricts how many generator tasks can be executed at once.
|
||||||
|
#[derive(Clone)]
|
||||||
|
struct PayloadTaskGuard(Arc<Semaphore>);
|
||||||
|
|
||||||
|
// === impl PayloadTaskGuard ===
|
||||||
|
|
||||||
|
impl PayloadTaskGuard {
|
||||||
|
fn new(max_payload_tasks: usize) -> Self {
|
||||||
|
Self(Arc::new(Semaphore::new(max_payload_tasks)))
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
/// Settings for the [BasicPayloadJobGenerator].
|
||||||
|
#[derive(Debug, Clone)]
|
||||||
|
pub struct BasicPayloadJobGeneratorConfig {
|
||||||
|
/// The interval at which the job should build a new payload after the last.
|
||||||
|
interval: Duration,
|
||||||
|
/// The deadline when this job should resolve.
|
||||||
|
deadline: Duration,
|
||||||
|
/// Maximum number of tasks to spawn for building a payload.
|
||||||
|
max_payload_tasks: usize,
|
||||||
|
}
|
||||||
|
|
||||||
|
// === impl BasicPayloadJobGeneratorConfig ===
|
||||||
|
|
||||||
|
impl BasicPayloadJobGeneratorConfig {
|
||||||
|
/// Sets the interval at which the job should build a new payload after the last.
|
||||||
|
pub fn interval(mut self, interval: Duration) -> Self {
|
||||||
|
self.interval = interval;
|
||||||
|
self
|
||||||
|
}
|
||||||
|
|
||||||
|
/// Sets the deadline when this job should resolve.
|
||||||
|
pub fn deadline(mut self, deadline: Duration) -> Self {
|
||||||
|
self.deadline = deadline;
|
||||||
|
self
|
||||||
|
}
|
||||||
|
|
||||||
|
/// Sets the maximum number of tasks to spawn for building a payload(s).
|
||||||
|
///
|
||||||
|
/// # Panics
|
||||||
|
///
|
||||||
|
/// If `max_payload_tasks` is 0.
|
||||||
|
pub fn max_payload_tasks(mut self, max_payload_tasks: usize) -> Self {
|
||||||
|
assert!(max_payload_tasks > 0, "max_payload_tasks must be greater than 0");
|
||||||
|
self.max_payload_tasks = max_payload_tasks;
|
||||||
|
self
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
impl Default for BasicPayloadJobGeneratorConfig {
|
||||||
|
fn default() -> Self {
|
||||||
|
Self {
|
||||||
|
interval: Duration::from_secs(1),
|
||||||
|
// 12s slot time
|
||||||
|
deadline: Duration::from_secs(12),
|
||||||
|
max_payload_tasks: 3,
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
/// A basic payload job that continuously builds a payload with the best transactions from the pool.
|
||||||
|
pub struct BasicPayloadJob<Client, Pool, Tasks> {
|
||||||
|
/// The configuration for how the payload will be created.
|
||||||
|
config: PayloadConfig,
|
||||||
|
/// The client that can interact with the chain.
|
||||||
|
client: Client,
|
||||||
|
/// The transaction pool.
|
||||||
|
pool: Pool,
|
||||||
|
/// How to spawn building tasks
|
||||||
|
executor: Tasks,
|
||||||
|
/// The deadline when this job should resolve.
|
||||||
|
deadline: Pin<Box<Sleep>>,
|
||||||
|
/// The interval at which the job should build a new payload after the last.
|
||||||
|
interval: Interval,
|
||||||
|
/// The best payload so far.
|
||||||
|
best_payload: Option<Arc<BuiltPayload>>,
|
||||||
|
/// Receiver for the block that is currently being built.
|
||||||
|
pending_block: Option<PendingPayload>,
|
||||||
|
/// Restricts how many generator tasks can be executed at once.
|
||||||
|
payload_task_guard: PayloadTaskGuard,
|
||||||
|
}
|
||||||
|
|
||||||
|
// === impl BasicPayloadJob ===
|
||||||
|
|
||||||
|
impl<Client, Pool, Tasks> BasicPayloadJob<Client, Pool, Tasks> {
|
||||||
|
/// Checks if the new payload is better than the current best.
|
||||||
|
///
|
||||||
|
/// This compares the total fees of the blocks, higher is better.
|
||||||
|
fn is_better(&self, new_payload: &BuiltPayload) -> bool {
|
||||||
|
if let Some(best_payload) = &self.best_payload {
|
||||||
|
new_payload.fees() > best_payload.fees()
|
||||||
|
} else {
|
||||||
|
true
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
impl<Client, Pool, Tasks> Stream for BasicPayloadJob<Client, Pool, Tasks>
|
||||||
|
where
|
||||||
|
Client: StateProviderFactory + Clone + Unpin + 'static,
|
||||||
|
Pool: TransactionPool + Unpin + 'static,
|
||||||
|
Tasks: TaskSpawner + Clone + 'static,
|
||||||
|
{
|
||||||
|
type Item = Result<Arc<BuiltPayload>, PayloadBuilderError>;
|
||||||
|
|
||||||
|
fn poll_next(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
|
||||||
|
let this = self.get_mut();
|
||||||
|
|
||||||
|
// check if the deadline is reached
|
||||||
|
let deadline_reached = this.deadline.as_mut().poll(cx).is_ready();
|
||||||
|
|
||||||
|
// check if the interval is reached
|
||||||
|
if this.interval.poll_tick(cx).is_ready() &&
|
||||||
|
this.pending_block.is_none() &&
|
||||||
|
!deadline_reached
|
||||||
|
{
|
||||||
|
trace!("spawn new payload build task");
|
||||||
|
let (tx, rx) = oneshot::channel();
|
||||||
|
let client = this.client.clone();
|
||||||
|
let pool = this.pool.clone();
|
||||||
|
let cancel = Cancelled::default();
|
||||||
|
let _cancel = cancel.clone();
|
||||||
|
let guard = this.payload_task_guard.clone();
|
||||||
|
let payload_config = this.config.clone();
|
||||||
|
this.executor.spawn_blocking(Box::pin(async move {
|
||||||
|
// acquire the permit for executing the task
|
||||||
|
let _permit = guard.0.acquire().await;
|
||||||
|
build_payload(client, pool, payload_config, cancel, tx)
|
||||||
|
}));
|
||||||
|
this.pending_block = Some(PendingPayload { _cancel, payload: rx });
|
||||||
|
}
|
||||||
|
|
||||||
|
// poll the pending block
|
||||||
|
if let Some(mut fut) = this.pending_block.take() {
|
||||||
|
match fut.poll_unpin(cx) {
|
||||||
|
Poll::Ready(Ok(payload)) => {
|
||||||
|
this.interval.reset();
|
||||||
|
if this.is_better(&payload) {
|
||||||
|
let payload = Arc::new(payload);
|
||||||
|
this.best_payload = Some(payload.clone());
|
||||||
|
return Poll::Ready(Some(Ok(payload)))
|
||||||
|
}
|
||||||
|
}
|
||||||
|
Poll::Ready(Err(err)) => {
|
||||||
|
this.interval.reset();
|
||||||
|
return Poll::Ready(Some(Err(err)))
|
||||||
|
}
|
||||||
|
Poll::Pending => {
|
||||||
|
this.pending_block = Some(fut);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
if deadline_reached {
|
||||||
|
trace!("Payload building deadline reached");
|
||||||
|
return Poll::Ready(None)
|
||||||
|
}
|
||||||
|
|
||||||
|
Poll::Pending
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
impl<Client, Pool, Tasks> PayloadJob for BasicPayloadJob<Client, Pool, Tasks>
|
||||||
|
where
|
||||||
|
Client: StateProviderFactory + Clone + Unpin + 'static,
|
||||||
|
Pool: TransactionPool + Unpin + 'static,
|
||||||
|
Tasks: TaskSpawner + Clone + 'static,
|
||||||
|
{
|
||||||
|
fn best_payload(&self) -> Arc<BuiltPayload> {
|
||||||
|
// TODO if still not set, initialize empty block
|
||||||
|
self.best_payload.clone().unwrap()
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
/// A future that resolves to the result of the block building job.
|
||||||
|
struct PendingPayload {
|
||||||
|
/// The marker to cancel the job on drop
|
||||||
|
_cancel: Cancelled,
|
||||||
|
/// The channel to send the result to.
|
||||||
|
payload: oneshot::Receiver<Result<BuiltPayload, PayloadBuilderError>>,
|
||||||
|
}
|
||||||
|
|
||||||
|
impl Future for PendingPayload {
|
||||||
|
type Output = Result<BuiltPayload, PayloadBuilderError>;
|
||||||
|
|
||||||
|
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
|
||||||
|
let res = ready!(self.payload.poll_unpin(cx));
|
||||||
|
Poll::Ready(res.map_err(Into::into).and_then(|res| res))
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
/// A marker that can be used to cancel a job.
|
||||||
|
#[derive(Default, Clone)]
|
||||||
|
struct Cancelled(Arc<AtomicBool>);
|
||||||
|
|
||||||
|
// === impl Cancelled ===
|
||||||
|
|
||||||
|
impl Cancelled {
|
||||||
|
/// Returns true if the job was cancelled.
|
||||||
|
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)]
|
||||||
|
struct PayloadConfig {
|
||||||
|
/// Pre-configured block environment.
|
||||||
|
initialized_block_env: BlockEnv,
|
||||||
|
/// Configuration for the environment.
|
||||||
|
initialized_cfg: CfgEnv,
|
||||||
|
/// The parent block.
|
||||||
|
parent_block: Arc<Block>,
|
||||||
|
/// Block extra data.
|
||||||
|
extra_data: Bytes,
|
||||||
|
/// Requested attributes for the payload.
|
||||||
|
attributes: PayloadBuilderAttributes,
|
||||||
|
/// The chain spec.
|
||||||
|
chain_spec: Arc<ChainSpec>,
|
||||||
|
}
|
||||||
|
|
||||||
|
/// Builds the payload and sends the result to the given channel.
|
||||||
|
fn build_payload<Pool, Client>(
|
||||||
|
client: Client,
|
||||||
|
pool: Pool,
|
||||||
|
config: PayloadConfig,
|
||||||
|
cancel: Cancelled,
|
||||||
|
to_job: oneshot::Sender<Result<BuiltPayload, PayloadBuilderError>>,
|
||||||
|
) where
|
||||||
|
Client: StateProviderFactory,
|
||||||
|
Pool: TransactionPool,
|
||||||
|
{
|
||||||
|
#[inline(always)]
|
||||||
|
fn try_build<Pool, Client>(
|
||||||
|
client: Client,
|
||||||
|
pool: Pool,
|
||||||
|
config: PayloadConfig,
|
||||||
|
cancel: Cancelled,
|
||||||
|
) -> Result<BuiltPayload, PayloadBuilderError>
|
||||||
|
where
|
||||||
|
Client: StateProviderFactory,
|
||||||
|
Pool: TransactionPool,
|
||||||
|
{
|
||||||
|
let PayloadConfig {
|
||||||
|
initialized_block_env,
|
||||||
|
initialized_cfg,
|
||||||
|
parent_block,
|
||||||
|
extra_data,
|
||||||
|
attributes,
|
||||||
|
chain_spec,
|
||||||
|
} = config;
|
||||||
|
|
||||||
|
// TODO this needs to access the _pending_ state of the parent block hash
|
||||||
|
let state = client.latest()?;
|
||||||
|
|
||||||
|
let mut db = SubState::new(State::new(state));
|
||||||
|
let mut post_state = PostState::default();
|
||||||
|
|
||||||
|
let mut cumulative_gas_used = 0;
|
||||||
|
let block_gas_limit: u64 = initialized_block_env.gas_limit.try_into().unwrap_or(u64::MAX);
|
||||||
|
|
||||||
|
let mut executed_txs = Vec::new();
|
||||||
|
let best_txs = pool.best_transactions();
|
||||||
|
|
||||||
|
let mut total_fees = U256::ZERO;
|
||||||
|
let base_fee = initialized_block_env.basefee.to::<u64>();
|
||||||
|
|
||||||
|
for tx in best_txs {
|
||||||
|
// ensure we still have capacity for this transaction
|
||||||
|
if cumulative_gas_used + tx.gas_limit() > block_gas_limit {
|
||||||
|
// TODO: try find transactions that can fit into the block
|
||||||
|
break
|
||||||
|
}
|
||||||
|
|
||||||
|
// check if the job was cancelled, if so we can exit early
|
||||||
|
if cancel.is_cancelled() {
|
||||||
|
return Err(PayloadBuilderError::BuildJobCancelled)
|
||||||
|
}
|
||||||
|
|
||||||
|
// convert tx to a signed transaction
|
||||||
|
let tx = tx.to_recovered_transaction();
|
||||||
|
|
||||||
|
// Configure the environment for the block.
|
||||||
|
let env = Env {
|
||||||
|
cfg: initialized_cfg.clone(),
|
||||||
|
block: initialized_block_env.clone(),
|
||||||
|
tx: tx_env_with_recovered(&tx),
|
||||||
|
};
|
||||||
|
|
||||||
|
let mut evm = revm::EVM::with_env(env);
|
||||||
|
evm.database(&mut db);
|
||||||
|
|
||||||
|
// TODO skip invalid transactions
|
||||||
|
let ResultAndState { result, state } =
|
||||||
|
evm.transact().map_err(PayloadBuilderError::EvmExecutionError)?;
|
||||||
|
|
||||||
|
// commit changes
|
||||||
|
commit_state_changes(&mut db, &mut post_state, state, true);
|
||||||
|
|
||||||
|
// Push transaction changeset and calculate header bloom filter for receipt.
|
||||||
|
post_state.add_receipt(Receipt {
|
||||||
|
tx_type: tx.tx_type(),
|
||||||
|
success: result.is_success(),
|
||||||
|
cumulative_gas_used,
|
||||||
|
logs: result.logs().into_iter().map(into_reth_log).collect(),
|
||||||
|
});
|
||||||
|
|
||||||
|
let gas_used = result.gas_used();
|
||||||
|
|
||||||
|
// update add to total fees
|
||||||
|
let miner_fee = tx
|
||||||
|
.effective_tip_per_gas(base_fee)
|
||||||
|
.expect("fee is always valid; execution succeeded");
|
||||||
|
total_fees += U256::from(miner_fee) * U256::from(gas_used);
|
||||||
|
|
||||||
|
// append gas used
|
||||||
|
cumulative_gas_used += gas_used;
|
||||||
|
|
||||||
|
// append transaction to the list of executed transactions
|
||||||
|
executed_txs.push(tx.into_signed());
|
||||||
|
}
|
||||||
|
|
||||||
|
let mut withdrawals_root = None;
|
||||||
|
|
||||||
|
// get balance changes from withdrawals
|
||||||
|
if initialized_cfg.spec_id >= SpecId::SHANGHAI {
|
||||||
|
let balance_increments = post_block_withdrawals_balance_increments(
|
||||||
|
&chain_spec,
|
||||||
|
attributes.timestamp,
|
||||||
|
&attributes.withdrawals,
|
||||||
|
);
|
||||||
|
for (address, increment) in balance_increments {
|
||||||
|
increment_account_balance(&mut db, &mut post_state, address, increment)?;
|
||||||
|
}
|
||||||
|
|
||||||
|
// calculate withdrawals root
|
||||||
|
withdrawals_root =
|
||||||
|
Some(proofs::calculate_withdrawals_root(attributes.withdrawals.iter()));
|
||||||
|
}
|
||||||
|
|
||||||
|
// create the block header
|
||||||
|
let transactions_root = proofs::calculate_transaction_root(executed_txs.iter());
|
||||||
|
|
||||||
|
let receipts_root = post_state.receipts_root();
|
||||||
|
let logs_bloom = post_state.logs_bloom();
|
||||||
|
|
||||||
|
let header = Header {
|
||||||
|
parent_hash: attributes.parent,
|
||||||
|
ommers_hash: EMPTY_OMMER_ROOT,
|
||||||
|
beneficiary: initialized_block_env.coinbase,
|
||||||
|
// TODO compute state root
|
||||||
|
state_root: Default::default(),
|
||||||
|
transactions_root,
|
||||||
|
receipts_root,
|
||||||
|
withdrawals_root,
|
||||||
|
logs_bloom,
|
||||||
|
timestamp: attributes.timestamp,
|
||||||
|
mix_hash: attributes.prev_randao,
|
||||||
|
nonce: 0,
|
||||||
|
base_fee_per_gas: Some(base_fee),
|
||||||
|
number: parent_block.number + 1,
|
||||||
|
gas_limit: block_gas_limit,
|
||||||
|
difficulty: U256::ZERO,
|
||||||
|
gas_used: cumulative_gas_used,
|
||||||
|
extra_data: extra_data.into(),
|
||||||
|
};
|
||||||
|
|
||||||
|
// seal the block
|
||||||
|
let block = Block {
|
||||||
|
header,
|
||||||
|
body: executed_txs,
|
||||||
|
ommers: vec![],
|
||||||
|
withdrawals: Some(attributes.withdrawals),
|
||||||
|
};
|
||||||
|
|
||||||
|
let sealed_block = block.seal_slow();
|
||||||
|
Ok(BuiltPayload::new(attributes.id, sealed_block, total_fees))
|
||||||
|
}
|
||||||
|
let _ = to_job.send(try_build(client, pool, config, cancel));
|
||||||
|
}
|
||||||
@ -1,6 +1,6 @@
|
|||||||
use crate::{
|
use crate::{
|
||||||
keccak256, Address, Bytes, GenesisAccount, Header, Log, ReceiptWithBloom, TransactionSigned,
|
keccak256, Address, Bytes, GenesisAccount, Header, Log, ReceiptWithBloom, ReceiptWithBloomRef,
|
||||||
Withdrawal, H256,
|
TransactionSigned, Withdrawal, H256,
|
||||||
};
|
};
|
||||||
use bytes::BytesMut;
|
use bytes::BytesMut;
|
||||||
use hash_db::Hasher;
|
use hash_db::Hasher;
|
||||||
@ -68,6 +68,17 @@ pub fn calculate_receipt_root<'a>(receipts: impl Iterator<Item = &'a ReceiptWith
|
|||||||
}))
|
}))
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/// Calculates the receipt root for a header for the reference type of [ReceiptWithBloom].
|
||||||
|
pub fn calculate_receipt_root_ref<'a>(
|
||||||
|
receipts: impl Iterator<Item = ReceiptWithBloomRef<'a>>,
|
||||||
|
) -> H256 {
|
||||||
|
ordered_trie_root::<KeccakHasher, _>(receipts.into_iter().map(|receipt| {
|
||||||
|
let mut receipt_rlp = Vec::new();
|
||||||
|
receipt.encode_inner(&mut receipt_rlp, false);
|
||||||
|
receipt_rlp
|
||||||
|
}))
|
||||||
|
}
|
||||||
|
|
||||||
/// Calculates the log root for headers.
|
/// Calculates the log root for headers.
|
||||||
pub fn calculate_log_root<'a>(logs: impl Iterator<Item = &'a Log> + Clone) -> H256 {
|
pub fn calculate_log_root<'a>(logs: impl Iterator<Item = &'a Log> + Clone) -> H256 {
|
||||||
//https://github.com/ethereum/go-ethereum/blob/356bbe343a30789e77bb38f25983c8f2f2bfbb47/cmd/evm/internal/t8ntool/execution.go#L255
|
//https://github.com/ethereum/go-ethereum/blob/356bbe343a30789e77bb38f25983c8f2f2bfbb47/cmd/evm/internal/t8ntool/execution.go#L255
|
||||||
|
|||||||
@ -164,6 +164,11 @@ impl<'a> ReceiptWithBloomRef<'a> {
|
|||||||
Self { receipt, bloom }
|
Self { receipt, bloom }
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/// Encode receipt with or without the header data.
|
||||||
|
pub fn encode_inner(&self, out: &mut dyn BufMut, with_header: bool) {
|
||||||
|
self.as_encoder().encode_inner(out, with_header)
|
||||||
|
}
|
||||||
|
|
||||||
#[inline]
|
#[inline]
|
||||||
fn as_encoder(&self) -> ReceiptWithBloomEncoder<'_> {
|
fn as_encoder(&self) -> ReceiptWithBloomEncoder<'_> {
|
||||||
ReceiptWithBloomEncoder { receipt: self.receipt, bloom: &self.bloom }
|
ReceiptWithBloomEncoder { receipt: self.receipt, bloom: &self.bloom }
|
||||||
|
|||||||
@ -17,7 +17,7 @@ reth-executor = { path = "../executor" }
|
|||||||
reth-consensus-common = { path = "../consensus/common" }
|
reth-consensus-common = { path = "../consensus/common" }
|
||||||
|
|
||||||
# revm
|
# revm
|
||||||
revm = { version = "3.1.0" }
|
revm = { version = "3" }
|
||||||
|
|
||||||
# common
|
# common
|
||||||
tracing = "0.1.37"
|
tracing = "0.1.37"
|
||||||
|
|||||||
@ -2,6 +2,21 @@
|
|||||||
|
|
||||||
use reth_primitives::{ChainSpec, Hardfork, Head};
|
use reth_primitives::{ChainSpec, Hardfork, Head};
|
||||||
|
|
||||||
|
/// Returns the spec id at the given timestamp.
|
||||||
|
///
|
||||||
|
/// Note: This is only intended to be used after the merge, when hardforks are activated by
|
||||||
|
/// timestamp.
|
||||||
|
pub fn revm_spec_by_timestamp_after_merge(
|
||||||
|
chain_spec: &ChainSpec,
|
||||||
|
timestamp: u64,
|
||||||
|
) -> revm::primitives::SpecId {
|
||||||
|
if chain_spec.is_fork_active_at_timestamp(Hardfork::Shanghai, timestamp) {
|
||||||
|
revm::primitives::SHANGHAI
|
||||||
|
} else {
|
||||||
|
revm::primitives::MERGE
|
||||||
|
}
|
||||||
|
}
|
||||||
|
|
||||||
/// return revm_spec from spec configuration.
|
/// return revm_spec from spec configuration.
|
||||||
pub fn revm_spec(chain_spec: &ChainSpec, block: Head) -> revm::primitives::SpecId {
|
pub fn revm_spec(chain_spec: &ChainSpec, block: Head) -> revm::primitives::SpecId {
|
||||||
if chain_spec.fork(Hardfork::Shanghai).active_at_head(&block) {
|
if chain_spec.fork(Hardfork::Shanghai).active_at_head(&block) {
|
||||||
|
|||||||
@ -108,7 +108,7 @@ where
|
|||||||
post_state: &mut PostState,
|
post_state: &mut PostState,
|
||||||
) {
|
) {
|
||||||
let db = self.db();
|
let db = self.db();
|
||||||
commit_changes(db, changes, has_state_clear_eip, post_state);
|
commit_state_changes(db, post_state, changes, has_state_clear_eip);
|
||||||
}
|
}
|
||||||
|
|
||||||
/// Collect all balance changes at the end of the block.
|
/// Collect all balance changes at the end of the block.
|
||||||
@ -371,11 +371,11 @@ where
|
|||||||
///
|
///
|
||||||
/// Note: This does _not_ commit to the underlying database [DatabaseRef], but only to the
|
/// Note: This does _not_ commit to the underlying database [DatabaseRef], but only to the
|
||||||
/// [CacheDB].
|
/// [CacheDB].
|
||||||
pub fn commit_changes<DB>(
|
pub fn commit_state_changes<DB>(
|
||||||
db: &mut CacheDB<DB>,
|
db: &mut CacheDB<DB>,
|
||||||
|
post_state: &mut PostState,
|
||||||
changes: hash_map::HashMap<Address, RevmAccount>,
|
changes: hash_map::HashMap<Address, RevmAccount>,
|
||||||
has_state_clear_eip: bool,
|
has_state_clear_eip: bool,
|
||||||
post_state: &mut PostState,
|
|
||||||
) where
|
) where
|
||||||
DB: DatabaseRef,
|
DB: DatabaseRef,
|
||||||
{
|
{
|
||||||
|
|||||||
@ -7,8 +7,8 @@ use reth_db::{
|
|||||||
Error as DbError,
|
Error as DbError,
|
||||||
};
|
};
|
||||||
use reth_primitives::{
|
use reth_primitives::{
|
||||||
bloom::logs_bloom, Account, Address, Bloom, Bytecode, Log, Receipt, StorageEntry, TransitionId,
|
bloom::logs_bloom, proofs::calculate_receipt_root_ref, Account, Address, Bloom, Bytecode, Log,
|
||||||
H256, U256,
|
Receipt, StorageEntry, TransitionId, H256, U256,
|
||||||
};
|
};
|
||||||
use std::collections::BTreeMap;
|
use std::collections::BTreeMap;
|
||||||
|
|
||||||
@ -273,6 +273,11 @@ impl PostState {
|
|||||||
logs_bloom(self.logs())
|
logs_bloom(self.logs())
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/// Returns the receipt root for all recorded receipts.
|
||||||
|
pub fn receipts_root(&self) -> H256 {
|
||||||
|
calculate_receipt_root_ref(self.receipts().iter().map(Into::into))
|
||||||
|
}
|
||||||
|
|
||||||
/// Get the number of transitions causing this [PostState]
|
/// Get the number of transitions causing this [PostState]
|
||||||
pub fn transitions_count(&self) -> TransitionId {
|
pub fn transitions_count(&self) -> TransitionId {
|
||||||
self.current_transition_id
|
self.current_transition_id
|
||||||
|
|||||||
@ -11,6 +11,14 @@ pub trait EvmEnvProvider: Send + Sync {
|
|||||||
/// Fills the [CfgEnv] and [BlockEnv] fields with values specific to the given [BlockId].
|
/// Fills the [CfgEnv] and [BlockEnv] fields with values specific to the given [BlockId].
|
||||||
fn fill_env_at(&self, cfg: &mut CfgEnv, block_env: &mut BlockEnv, at: BlockId) -> Result<()>;
|
fn fill_env_at(&self, cfg: &mut CfgEnv, block_env: &mut BlockEnv, at: BlockId) -> Result<()>;
|
||||||
|
|
||||||
|
/// Fills the default [CfgEnv] and [BlockEnv] fields with values specific to the given [Header].
|
||||||
|
fn env_with_header(&self, header: &Header) -> Result<(CfgEnv, BlockEnv)> {
|
||||||
|
let mut cfg = CfgEnv::default();
|
||||||
|
let mut block_env = BlockEnv::default();
|
||||||
|
self.fill_env_with_header(&mut cfg, &mut block_env, header)?;
|
||||||
|
Ok((cfg, block_env))
|
||||||
|
}
|
||||||
|
|
||||||
/// Fills the [CfgEnv] and [BlockEnv] fields with values specific to the given [Header].
|
/// Fills the [CfgEnv] and [BlockEnv] fields with values specific to the given [Header].
|
||||||
fn fill_env_with_header(
|
fn fill_env_with_header(
|
||||||
&self,
|
&self,
|
||||||
|
|||||||
@ -75,7 +75,7 @@ pub trait TaskSpawner: Send + Sync + Unpin + std::fmt::Debug + DynClone {
|
|||||||
fn spawn_critical(&self, name: &'static str, fut: BoxFuture<'static, ()>) -> JoinHandle<()>;
|
fn spawn_critical(&self, name: &'static str, fut: BoxFuture<'static, ()>) -> JoinHandle<()>;
|
||||||
|
|
||||||
/// Spawns a blocking task onto the runtime.
|
/// Spawns a blocking task onto the runtime.
|
||||||
fn spawn_blocking(&self, name: &'static str, fut: BoxFuture<'static, ()>) -> JoinHandle<()>;
|
fn spawn_blocking(&self, fut: BoxFuture<'static, ()>) -> JoinHandle<()>;
|
||||||
}
|
}
|
||||||
|
|
||||||
dyn_clone::clone_trait_object!(TaskSpawner);
|
dyn_clone::clone_trait_object!(TaskSpawner);
|
||||||
@ -94,7 +94,7 @@ impl TaskSpawner for TokioTaskExecutor {
|
|||||||
tokio::task::spawn(fut)
|
tokio::task::spawn(fut)
|
||||||
}
|
}
|
||||||
|
|
||||||
fn spawn_blocking(&self, _name: &'static str, fut: BoxFuture<'static, ()>) -> JoinHandle<()> {
|
fn spawn_blocking(&self, fut: BoxFuture<'static, ()>) -> JoinHandle<()> {
|
||||||
tokio::task::spawn_blocking(move || tokio::runtime::Handle::current().block_on(fut))
|
tokio::task::spawn_blocking(move || tokio::runtime::Handle::current().block_on(fut))
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
@ -350,7 +350,7 @@ impl TaskSpawner for TaskExecutor {
|
|||||||
TaskExecutor::spawn_critical(self, name, fut)
|
TaskExecutor::spawn_critical(self, name, fut)
|
||||||
}
|
}
|
||||||
|
|
||||||
fn spawn_blocking(&self, _name: &'static str, fut: BoxFuture<'static, ()>) -> JoinHandle<()> {
|
fn spawn_blocking(&self, fut: BoxFuture<'static, ()>) -> JoinHandle<()> {
|
||||||
self.spawn_blocking(fut)
|
self.spawn_blocking(fut)
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|||||||
Reference in New Issue
Block a user