mirror of
https://github.com/hl-archive-node/nanoreth.git
synced 2025-12-06 19:09:54 +00:00
feat: add BuiltPayload associated type to EngineTypes (#6028)
This commit is contained in:
@ -142,6 +142,7 @@ pub trait RethNodeCommandConfig: fmt::Debug {
|
||||
Reth::Pool,
|
||||
Reth::Provider,
|
||||
Attributes = Engine::PayloadBuilderAttributes,
|
||||
BuiltPayload = Engine::BuiltPayload,
|
||||
> + Unpin
|
||||
+ 'static,
|
||||
{
|
||||
@ -331,6 +332,7 @@ impl<T: RethNodeCommandConfig> RethNodeCommandConfig for NoArgs<T> {
|
||||
Reth::Pool,
|
||||
Reth::Provider,
|
||||
Attributes = Engine::PayloadBuilderAttributes,
|
||||
BuiltPayload = Engine::BuiltPayload,
|
||||
> + Unpin
|
||||
+ 'static,
|
||||
{
|
||||
|
||||
@ -18,7 +18,7 @@
|
||||
//! # use reth_rpc_types::engine::{PayloadAttributes as EthPayloadAttributes, PayloadId, Withdrawal};
|
||||
//! # use reth_primitives::{B256, ChainSpec, Address};
|
||||
//! # use reth_node_api::{EngineTypes, EngineApiMessageVersion, validate_version_specific_fields, AttributesValidationError, PayloadAttributes, PayloadBuilderAttributes};
|
||||
//! # use reth_payload_builder::EthPayloadBuilderAttributes;
|
||||
//! # use reth_payload_builder::{EthPayloadBuilderAttributes, EthBuiltPayload};
|
||||
//! # use serde::{Deserialize, Serialize};
|
||||
//! # use thiserror::Error;
|
||||
//! # use std::convert::Infallible;
|
||||
@ -121,6 +121,7 @@
|
||||
//! impl EngineTypes for CustomEngineTypes {
|
||||
//! type PayloadAttributes = CustomPayloadAttributes;
|
||||
//! type PayloadBuilderAttributes = CustomPayloadBuilderAttributes;
|
||||
//! type BuiltPayload = EthBuiltPayload;
|
||||
//! }
|
||||
//! ```
|
||||
|
||||
@ -129,7 +130,7 @@ use reth_primitives::{ChainSpec, Hardfork};
|
||||
/// Contains traits to abstract over payload attributes types and default implementations of the
|
||||
/// [PayloadAttributes] trait for ethereum mainnet and optimism types.
|
||||
pub mod traits;
|
||||
pub use traits::{PayloadAttributes, PayloadBuilderAttributes};
|
||||
pub use traits::{BuiltPayload, PayloadAttributes, PayloadBuilderAttributes};
|
||||
|
||||
/// Contains error types used in the traits defined in this crate.
|
||||
pub mod error;
|
||||
@ -149,7 +150,8 @@ pub trait EngineTypes: Send + Sync {
|
||||
+ Clone
|
||||
+ Unpin;
|
||||
|
||||
// TODO: payload type
|
||||
/// The built payload type.
|
||||
type BuiltPayload: BuiltPayload + Clone + Unpin;
|
||||
}
|
||||
|
||||
/// Validates the timestamp depending on the version called:
|
||||
|
||||
@ -2,12 +2,38 @@ use crate::{validate_version_specific_fields, AttributesValidationError, EngineA
|
||||
use reth_primitives::{
|
||||
revm::config::revm_spec_by_timestamp_after_merge,
|
||||
revm_primitives::{BlobExcessGasAndPrice, BlockEnv, CfgEnv, SpecId},
|
||||
Address, ChainSpec, Header, B256, U256,
|
||||
Address, ChainSpec, Header, SealedBlock, B256, U256,
|
||||
};
|
||||
use reth_rpc_types::engine::{
|
||||
OptimismPayloadAttributes, PayloadAttributes as EthPayloadAttributes, PayloadId, Withdrawal,
|
||||
use reth_rpc_types::{
|
||||
engine::{
|
||||
ExecutionPayloadEnvelopeV2, ExecutionPayloadEnvelopeV3, OptimismPayloadAttributes,
|
||||
PayloadAttributes as EthPayloadAttributes, PayloadId, Withdrawal,
|
||||
},
|
||||
ExecutionPayloadV1,
|
||||
};
|
||||
|
||||
/// Represents a built payload type that contains a built [SealedBlock] and can be converted into
|
||||
/// engine API execution payloads.
|
||||
pub trait BuiltPayload: Send + Sync + std::fmt::Debug {
|
||||
/// Initializes the payload with the given initial block.
|
||||
fn new(id: PayloadId, block: SealedBlock, fees: U256) -> Self;
|
||||
|
||||
/// Returns the built block (sealed)
|
||||
fn block(&self) -> &SealedBlock;
|
||||
|
||||
/// Returns the fees collected for the built block
|
||||
fn fees(&self) -> U256;
|
||||
|
||||
/// Converts the type into the response expected by `engine_getPayloadV1`
|
||||
fn into_v1_payload(self) -> ExecutionPayloadV1;
|
||||
|
||||
/// Converts the type into the response expected by `engine_getPayloadV2`
|
||||
fn into_v2_payload(self) -> ExecutionPayloadEnvelopeV2;
|
||||
|
||||
/// Converts the type into the response expected by `engine_getPayloadV2`
|
||||
fn into_v3_payload(self) -> ExecutionPayloadEnvelopeV3;
|
||||
}
|
||||
|
||||
/// This can be implemented by types that describe a currently running payload job.
|
||||
///
|
||||
/// This is used as a conversion type, transforming a payload attributes type that the engine API
|
||||
|
||||
@ -13,6 +13,6 @@
|
||||
pub mod engine;
|
||||
pub use engine::{
|
||||
validate_payload_timestamp, validate_version_specific_fields, validate_withdrawals_presence,
|
||||
AttributesValidationError, EngineApiMessageVersion, EngineTypes, PayloadAttributes,
|
||||
PayloadBuilderAttributes, PayloadOrAttributes,
|
||||
AttributesValidationError, BuiltPayload, EngineApiMessageVersion, EngineTypes,
|
||||
PayloadAttributes, PayloadBuilderAttributes, PayloadOrAttributes,
|
||||
};
|
||||
|
||||
@ -1,5 +1,7 @@
|
||||
use reth_node_api::EngineTypes;
|
||||
use reth_payload_builder::{EthPayloadBuilderAttributes, OptimismPayloadBuilderAttributes};
|
||||
use reth_payload_builder::{
|
||||
EthBuiltPayload, EthPayloadBuilderAttributes, OptimismPayloadBuilderAttributes,
|
||||
};
|
||||
use reth_rpc_types::engine::{OptimismPayloadAttributes, PayloadAttributes};
|
||||
|
||||
/// The types used in the default mainnet ethereum beacon consensus engine.
|
||||
@ -10,6 +12,7 @@ pub struct EthEngineTypes;
|
||||
impl EngineTypes for EthEngineTypes {
|
||||
type PayloadAttributes = PayloadAttributes;
|
||||
type PayloadBuilderAttributes = EthPayloadBuilderAttributes;
|
||||
type BuiltPayload = EthBuiltPayload;
|
||||
}
|
||||
|
||||
/// The types used in the optimism beacon consensus engine.
|
||||
@ -20,4 +23,5 @@ pub struct OptimismEngineTypes;
|
||||
impl EngineTypes for OptimismEngineTypes {
|
||||
type PayloadAttributes = OptimismPayloadAttributes;
|
||||
type PayloadBuilderAttributes = OptimismPayloadBuilderAttributes;
|
||||
type BuiltPayload = EthBuiltPayload;
|
||||
}
|
||||
|
||||
@ -12,9 +12,9 @@ use alloy_rlp::Encodable;
|
||||
use futures_core::ready;
|
||||
use futures_util::FutureExt;
|
||||
use reth_interfaces::RethResult;
|
||||
use reth_node_api::PayloadBuilderAttributes;
|
||||
use reth_node_api::{BuiltPayload, PayloadBuilderAttributes};
|
||||
use reth_payload_builder::{
|
||||
database::CachedReads, error::PayloadBuilderError, BuiltPayload, KeepPayloadJobAlive,
|
||||
database::CachedReads, error::PayloadBuilderError, EthBuiltPayload, KeepPayloadJobAlive,
|
||||
PayloadId, PayloadJob, PayloadJobGenerator,
|
||||
};
|
||||
use reth_primitives::{
|
||||
@ -154,6 +154,7 @@ where
|
||||
Tasks: TaskSpawner + Clone + Unpin + 'static,
|
||||
Builder: PayloadBuilder<Pool, Client> + Unpin + 'static,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::Attributes: Unpin + Clone,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::BuiltPayload: Unpin + Clone,
|
||||
{
|
||||
type Job = BasicPayloadJob<Client, Pool, Tasks, Builder>;
|
||||
|
||||
@ -343,9 +344,9 @@ where
|
||||
/// 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>>,
|
||||
best_payload: Option<Builder::BuiltPayload>,
|
||||
/// Receiver for the block that is currently being built.
|
||||
pending_block: Option<PendingPayload>,
|
||||
pending_block: Option<PendingPayload<Builder::BuiltPayload>>,
|
||||
/// Restricts how many generator tasks can be executed at once.
|
||||
payload_task_guard: PayloadTaskGuard,
|
||||
/// Caches all disk reads for the state the new payloads builds on
|
||||
@ -368,6 +369,7 @@ where
|
||||
Tasks: TaskSpawner + Clone + 'static,
|
||||
Builder: PayloadBuilder<Pool, Client> + Unpin + 'static,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::Attributes: Unpin + Clone,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::BuiltPayload: Unpin + Clone,
|
||||
{
|
||||
type Output = Result<(), PayloadBuilderError>;
|
||||
|
||||
@ -424,7 +426,7 @@ where
|
||||
BuildOutcome::Better { payload, cached_reads } => {
|
||||
this.cached_reads = Some(cached_reads);
|
||||
debug!(target: "payload_builder", value = %payload.fees(), "built better payload");
|
||||
let payload = Arc::new(payload);
|
||||
let payload = payload;
|
||||
this.best_payload = Some(payload);
|
||||
}
|
||||
BuildOutcome::Aborted { fees, cached_reads } => {
|
||||
@ -457,12 +459,14 @@ where
|
||||
Pool: TransactionPool + Unpin + 'static,
|
||||
Tasks: TaskSpawner + Clone + 'static,
|
||||
Builder: PayloadBuilder<Pool, Client> + Unpin + 'static,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::Attributes: PayloadBuilderAttributes + Unpin + Clone,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::Attributes: Unpin + Clone,
|
||||
<Builder as PayloadBuilder<Pool, Client>>::BuiltPayload: Unpin + Clone,
|
||||
{
|
||||
type PayloadAttributes = Builder::Attributes;
|
||||
type ResolvePayloadFuture = ResolveBestPayload;
|
||||
type ResolvePayloadFuture = ResolveBestPayload<Self::BuiltPayload>;
|
||||
type BuiltPayload = Builder::BuiltPayload;
|
||||
|
||||
fn best_payload(&self) -> Result<Arc<BuiltPayload>, PayloadBuilderError> {
|
||||
fn best_payload(&self) -> Result<Self::BuiltPayload, PayloadBuilderError> {
|
||||
if let Some(ref payload) = self.best_payload {
|
||||
return Ok(payload.clone())
|
||||
}
|
||||
@ -473,7 +477,7 @@ where
|
||||
// away and the first full block should have been built by the time CL is requesting the
|
||||
// payload.
|
||||
self.metrics.inc_requested_empty_payload();
|
||||
build_empty_payload(&self.client, self.config.clone()).map(Arc::new)
|
||||
build_empty_payload(&self.client, self.config.clone())
|
||||
}
|
||||
|
||||
fn payload_attributes(&self) -> Result<Self::PayloadAttributes, PayloadBuilderError> {
|
||||
@ -538,17 +542,20 @@ where
|
||||
/// If no payload has been built so far, it will either return an empty payload or the result of the
|
||||
/// in progress build job, whatever finishes first.
|
||||
#[derive(Debug)]
|
||||
pub struct ResolveBestPayload {
|
||||
pub struct ResolveBestPayload<Payload> {
|
||||
/// Best payload so far.
|
||||
best_payload: Option<Arc<BuiltPayload>>,
|
||||
best_payload: Option<Payload>,
|
||||
/// Regular payload job that's currently running that might produce a better payload.
|
||||
maybe_better: Option<PendingPayload>,
|
||||
maybe_better: Option<PendingPayload<Payload>>,
|
||||
/// The empty payload building job in progress.
|
||||
empty_payload: Option<oneshot::Receiver<Result<BuiltPayload, PayloadBuilderError>>>,
|
||||
empty_payload: Option<oneshot::Receiver<Result<Payload, PayloadBuilderError>>>,
|
||||
}
|
||||
|
||||
impl Future for ResolveBestPayload {
|
||||
type Output = Result<Arc<BuiltPayload>, PayloadBuilderError>;
|
||||
impl<Payload> Future for ResolveBestPayload<Payload>
|
||||
where
|
||||
Payload: Unpin,
|
||||
{
|
||||
type Output = Result<Payload, PayloadBuilderError>;
|
||||
|
||||
fn poll(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
|
||||
let this = self.get_mut();
|
||||
@ -559,7 +566,7 @@ impl Future for ResolveBestPayload {
|
||||
this.maybe_better = None;
|
||||
if let Ok(BuildOutcome::Better { payload, .. }) = res {
|
||||
debug!(target: "payload_builder", "resolving better payload");
|
||||
return Poll::Ready(Ok(Arc::new(payload)))
|
||||
return Poll::Ready(Ok(payload))
|
||||
}
|
||||
}
|
||||
}
|
||||
@ -577,7 +584,7 @@ impl Future for ResolveBestPayload {
|
||||
} else {
|
||||
debug!(target: "payload_builder", "resolving empty payload");
|
||||
}
|
||||
Poll::Ready(res.map(Arc::new))
|
||||
Poll::Ready(res)
|
||||
}
|
||||
Poll::Ready(Err(err)) => Poll::Ready(Err(err.into())),
|
||||
Poll::Pending => {
|
||||
@ -590,15 +597,15 @@ impl Future for ResolveBestPayload {
|
||||
|
||||
/// A future that resolves to the result of the block building job.
|
||||
#[derive(Debug)]
|
||||
struct PendingPayload {
|
||||
struct PendingPayload<P> {
|
||||
/// The marker to cancel the job on drop
|
||||
_cancel: Cancelled,
|
||||
/// The channel to send the result to.
|
||||
payload: oneshot::Receiver<Result<BuildOutcome, PayloadBuilderError>>,
|
||||
payload: oneshot::Receiver<Result<BuildOutcome<P>, PayloadBuilderError>>,
|
||||
}
|
||||
|
||||
impl Future for PendingPayload {
|
||||
type Output = Result<BuildOutcome, PayloadBuilderError>;
|
||||
impl<P> Future for PendingPayload<P> {
|
||||
type Output = Result<BuildOutcome<P>, PayloadBuilderError>;
|
||||
|
||||
fn poll(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
|
||||
let res = ready!(self.payload.poll_unpin(cx));
|
||||
@ -684,11 +691,11 @@ where
|
||||
|
||||
/// The possible outcomes of a payload building attempt.
|
||||
#[derive(Debug)]
|
||||
pub enum BuildOutcome {
|
||||
pub enum BuildOutcome<Payload> {
|
||||
/// Successfully built a better block.
|
||||
Better {
|
||||
/// The new payload that was built.
|
||||
payload: BuiltPayload,
|
||||
payload: Payload,
|
||||
/// The cached reads that were used to build the payload.
|
||||
cached_reads: CachedReads,
|
||||
},
|
||||
@ -709,7 +716,7 @@ pub enum BuildOutcome {
|
||||
/// building process. It holds references to the Ethereum client, transaction pool, cached reads,
|
||||
/// payload configuration, cancellation status, and the best payload achieved so far.
|
||||
#[derive(Debug)]
|
||||
pub struct BuildArguments<Pool, Client, Attributes> {
|
||||
pub struct BuildArguments<Pool, Client, Attributes, Payload> {
|
||||
/// How to interact with the chain.
|
||||
pub client: Client,
|
||||
/// The transaction pool.
|
||||
@ -721,10 +728,10 @@ pub struct BuildArguments<Pool, Client, Attributes> {
|
||||
/// A marker that can be used to cancel the job.
|
||||
pub cancel: Cancelled,
|
||||
/// The best payload achieved so far.
|
||||
pub best_payload: Option<Arc<BuiltPayload>>,
|
||||
pub best_payload: Option<Payload>,
|
||||
}
|
||||
|
||||
impl<Pool, Client, Attributes> BuildArguments<Pool, Client, Attributes> {
|
||||
impl<Pool, Client, Attributes, Payload> BuildArguments<Pool, Client, Attributes, Payload> {
|
||||
/// Create new build arguments.
|
||||
pub fn new(
|
||||
client: Client,
|
||||
@ -732,7 +739,7 @@ impl<Pool, Client, Attributes> BuildArguments<Pool, Client, Attributes> {
|
||||
cached_reads: CachedReads,
|
||||
config: PayloadConfig<Attributes>,
|
||||
cancel: Cancelled,
|
||||
best_payload: Option<Arc<BuiltPayload>>,
|
||||
best_payload: Option<Payload>,
|
||||
) -> Self {
|
||||
Self { client, pool, cached_reads, config, cancel, best_payload }
|
||||
}
|
||||
@ -748,7 +755,9 @@ impl<Pool, Client, Attributes> BuildArguments<Pool, Client, Attributes> {
|
||||
/// Ethereum client types.
|
||||
pub trait PayloadBuilder<Pool, Client>: Send + Sync + Clone {
|
||||
/// The payload attributes type to accept for building.
|
||||
type Attributes: PayloadBuilderAttributes + Send + Sync + std::fmt::Debug;
|
||||
type Attributes: PayloadBuilderAttributes;
|
||||
/// The type of the built payload.
|
||||
type BuiltPayload: BuiltPayload;
|
||||
|
||||
/// Tries to build a transaction payload using provided arguments.
|
||||
///
|
||||
@ -764,8 +773,8 @@ pub trait PayloadBuilder<Pool, Client>: Send + Sync + Clone {
|
||||
/// A `Result` indicating the build outcome or an error.
|
||||
fn try_build(
|
||||
&self,
|
||||
args: BuildArguments<Pool, Client, Self::Attributes>,
|
||||
) -> Result<BuildOutcome, PayloadBuilderError>;
|
||||
args: BuildArguments<Pool, Client, Self::Attributes, Self::BuiltPayload>,
|
||||
) -> Result<BuildOutcome<Self::BuiltPayload>, PayloadBuilderError>;
|
||||
|
||||
/// Invoked when the payload job is being resolved and there is no payload yet.
|
||||
///
|
||||
@ -774,21 +783,22 @@ pub trait PayloadBuilder<Pool, Client>: Send + Sync + Clone {
|
||||
/// TODO(mattsse): This needs to be refined a bit because this only exists for OP atm
|
||||
fn on_missing_payload(
|
||||
&self,
|
||||
args: BuildArguments<Pool, Client, Self::Attributes>,
|
||||
) -> Option<Arc<BuiltPayload>> {
|
||||
args: BuildArguments<Pool, Client, Self::Attributes, Self::BuiltPayload>,
|
||||
) -> Option<Self::BuiltPayload> {
|
||||
let _args = args;
|
||||
None
|
||||
}
|
||||
}
|
||||
|
||||
/// Builds an empty payload without any transactions.
|
||||
fn build_empty_payload<Client, Attributes>(
|
||||
fn build_empty_payload<Client, Attributes, Payload>(
|
||||
client: &Client,
|
||||
config: PayloadConfig<Attributes>,
|
||||
) -> Result<BuiltPayload, PayloadBuilderError>
|
||||
) -> Result<Payload, PayloadBuilderError>
|
||||
where
|
||||
Client: StateProviderFactory,
|
||||
Attributes: PayloadBuilderAttributes,
|
||||
Payload: BuiltPayload,
|
||||
{
|
||||
let extra_data = config.extra_data();
|
||||
let PayloadConfig {
|
||||
@ -872,7 +882,7 @@ where
|
||||
let block = Block { header, body: vec![], ommers: vec![], withdrawals };
|
||||
let sealed_block = block.seal_slow();
|
||||
|
||||
Ok(BuiltPayload::new(attributes.payload_id(), sealed_block, U256::ZERO))
|
||||
Ok(Payload::new(attributes.payload_id(), sealed_block, U256::ZERO))
|
||||
}
|
||||
|
||||
/// Represents the outcome of committing withdrawals to the runtime database and post state.
|
||||
@ -978,7 +988,7 @@ where
|
||||
///
|
||||
/// This compares the total fees of the blocks, higher is better.
|
||||
#[inline(always)]
|
||||
pub fn is_better_payload(best_payload: Option<&BuiltPayload>, new_fees: U256) -> bool {
|
||||
pub fn is_better_payload(best_payload: Option<&EthBuiltPayload>, new_fees: U256) -> bool {
|
||||
if let Some(best_payload) = best_payload {
|
||||
new_fees > best_payload.fees()
|
||||
} else {
|
||||
|
||||
@ -26,9 +26,8 @@
|
||||
//! ```
|
||||
//! use std::future::Future;
|
||||
//! use std::pin::Pin;
|
||||
//! use std::sync::Arc;
|
||||
//! use std::task::{Context, Poll};
|
||||
//! use reth_payload_builder::{BuiltPayload, KeepPayloadJobAlive, EthPayloadBuilderAttributes, PayloadJob, PayloadJobGenerator};
|
||||
//! use reth_payload_builder::{EthBuiltPayload, KeepPayloadJobAlive, EthPayloadBuilderAttributes, PayloadJob, PayloadJobGenerator};
|
||||
//! use reth_payload_builder::error::PayloadBuilderError;
|
||||
//! use reth_primitives::{Block, Header, U256};
|
||||
//!
|
||||
@ -52,9 +51,10 @@
|
||||
//!
|
||||
//! impl PayloadJob for EmptyBlockPayloadJob {
|
||||
//! type PayloadAttributes = EthPayloadBuilderAttributes;
|
||||
//! type ResolvePayloadFuture = futures_util::future::Ready<Result<Arc<BuiltPayload>, PayloadBuilderError>>;
|
||||
//! type ResolvePayloadFuture = futures_util::future::Ready<Result<EthBuiltPayload, PayloadBuilderError>>;
|
||||
//! type BuiltPayload = EthBuiltPayload;
|
||||
//!
|
||||
//! fn best_payload(&self) -> Result<Arc<BuiltPayload>, PayloadBuilderError> {
|
||||
//! fn best_payload(&self) -> Result<EthBuiltPayload, PayloadBuilderError> {
|
||||
//! // NOTE: some fields are omitted here for brevity
|
||||
//! let payload = Block {
|
||||
//! header: Header {
|
||||
@ -65,8 +65,8 @@
|
||||
//! },
|
||||
//! ..Default::default()
|
||||
//! };
|
||||
//! let payload = BuiltPayload::new(self.attributes.id, payload.seal_slow(), U256::ZERO);
|
||||
//! Ok(Arc::new(payload))
|
||||
//! let payload = EthBuiltPayload::new(self.attributes.id, payload.seal_slow(), U256::ZERO);
|
||||
//! Ok(payload)
|
||||
//! }
|
||||
//!
|
||||
//! fn payload_attributes(&self) -> Result<EthPayloadBuilderAttributes, PayloadBuilderError> {
|
||||
@ -114,7 +114,7 @@ pub mod noop;
|
||||
pub mod test_utils;
|
||||
|
||||
pub use optimism::OptimismPayloadBuilderAttributes;
|
||||
pub use payload::{BuiltPayload, EthPayloadBuilderAttributes};
|
||||
pub use payload::{EthBuiltPayload, EthPayloadBuilderAttributes};
|
||||
pub use reth_rpc_types::engine::PayloadId;
|
||||
pub use service::{PayloadBuilderHandle, PayloadBuilderService, PayloadStore};
|
||||
pub use traits::{KeepPayloadJobAlive, PayloadJob, PayloadJobGenerator};
|
||||
|
||||
@ -15,7 +15,7 @@ use tokio_stream::wrappers::UnboundedReceiverStream;
|
||||
#[derive(Debug)]
|
||||
pub struct NoopPayloadBuilderService<Engine: EngineTypes> {
|
||||
/// Receiver half of the command channel.
|
||||
command_rx: UnboundedReceiverStream<PayloadServiceCommand<Engine::PayloadBuilderAttributes>>,
|
||||
command_rx: UnboundedReceiverStream<PayloadServiceCommand<Engine>>,
|
||||
}
|
||||
|
||||
impl<Engine> NoopPayloadBuilderService<Engine>
|
||||
|
||||
@ -1,7 +1,7 @@
|
||||
//! Contains types required for building a payload.
|
||||
|
||||
use alloy_rlp::Encodable;
|
||||
use reth_node_api::PayloadBuilderAttributes;
|
||||
use reth_node_api::{BuiltPayload, PayloadBuilderAttributes};
|
||||
use reth_primitives::{Address, BlobTransactionSidecar, SealedBlock, Withdrawal, B256, U256};
|
||||
use reth_rpc_types::engine::{
|
||||
ExecutionPayloadEnvelopeV2, ExecutionPayloadEnvelopeV3, ExecutionPayloadV1, PayloadAttributes,
|
||||
@ -19,7 +19,7 @@ use std::convert::Infallible;
|
||||
/// Therefore, the empty-block here is always available and full-block will be set/updated
|
||||
/// afterwards.
|
||||
#[derive(Debug, Clone)]
|
||||
pub struct BuiltPayload {
|
||||
pub struct EthBuiltPayload {
|
||||
/// Identifier of the payload
|
||||
pub(crate) id: PayloadId,
|
||||
/// The built block
|
||||
@ -33,7 +33,7 @@ pub struct BuiltPayload {
|
||||
|
||||
// === impl BuiltPayload ===
|
||||
|
||||
impl BuiltPayload {
|
||||
impl EthBuiltPayload {
|
||||
/// Initializes the payload with the given initial block.
|
||||
pub fn new(id: PayloadId, block: SealedBlock, fees: U256) -> Self {
|
||||
Self { id, block, fees, sidecars: Vec::new() }
|
||||
@ -75,17 +75,43 @@ impl BuiltPayload {
|
||||
}
|
||||
}
|
||||
|
||||
impl BuiltPayload for EthBuiltPayload {
|
||||
fn new(id: PayloadId, block: SealedBlock, fees: U256) -> Self {
|
||||
Self::new(id, block, fees)
|
||||
}
|
||||
|
||||
fn block(&self) -> &SealedBlock {
|
||||
&self.block
|
||||
}
|
||||
|
||||
fn fees(&self) -> U256 {
|
||||
self.fees
|
||||
}
|
||||
|
||||
fn into_v1_payload(self) -> ExecutionPayloadV1 {
|
||||
self.into()
|
||||
}
|
||||
|
||||
fn into_v2_payload(self) -> ExecutionPayloadEnvelopeV2 {
|
||||
self.into()
|
||||
}
|
||||
|
||||
fn into_v3_payload(self) -> ExecutionPayloadEnvelopeV3 {
|
||||
self.into()
|
||||
}
|
||||
}
|
||||
|
||||
// V1 engine_getPayloadV1 response
|
||||
impl From<BuiltPayload> for ExecutionPayloadV1 {
|
||||
fn from(value: BuiltPayload) -> Self {
|
||||
impl From<EthBuiltPayload> for ExecutionPayloadV1 {
|
||||
fn from(value: EthBuiltPayload) -> Self {
|
||||
try_block_to_payload_v1(value.block)
|
||||
}
|
||||
}
|
||||
|
||||
// V2 engine_getPayloadV2 response
|
||||
impl From<BuiltPayload> for ExecutionPayloadEnvelopeV2 {
|
||||
fn from(value: BuiltPayload) -> Self {
|
||||
let BuiltPayload { block, fees, .. } = value;
|
||||
impl From<EthBuiltPayload> for ExecutionPayloadEnvelopeV2 {
|
||||
fn from(value: EthBuiltPayload) -> Self {
|
||||
let EthBuiltPayload { block, fees, .. } = value;
|
||||
|
||||
ExecutionPayloadEnvelopeV2 {
|
||||
block_value: fees,
|
||||
@ -94,9 +120,9 @@ impl From<BuiltPayload> for ExecutionPayloadEnvelopeV2 {
|
||||
}
|
||||
}
|
||||
|
||||
impl From<BuiltPayload> for ExecutionPayloadEnvelopeV3 {
|
||||
fn from(value: BuiltPayload) -> Self {
|
||||
let BuiltPayload { block, fees, sidecars, .. } = value;
|
||||
impl From<EthBuiltPayload> for ExecutionPayloadEnvelopeV3 {
|
||||
fn from(value: EthBuiltPayload) -> Self {
|
||||
let EthBuiltPayload { block, fees, sidecars, .. } = value;
|
||||
|
||||
ExecutionPayloadEnvelopeV3 {
|
||||
execution_payload: block_to_payload_v3(block),
|
||||
|
||||
@ -5,17 +5,16 @@
|
||||
|
||||
use crate::{
|
||||
error::PayloadBuilderError, metrics::PayloadBuilderServiceMetrics, traits::PayloadJobGenerator,
|
||||
BuiltPayload, KeepPayloadJobAlive, PayloadJob,
|
||||
KeepPayloadJobAlive, PayloadJob,
|
||||
};
|
||||
use futures_util::{future::FutureExt, Stream, StreamExt};
|
||||
use reth_node_api::{EngineTypes, PayloadBuilderAttributes};
|
||||
use reth_node_api::{BuiltPayload, EngineTypes, PayloadBuilderAttributes};
|
||||
use reth_provider::CanonStateNotification;
|
||||
use reth_rpc_types::engine::PayloadId;
|
||||
use std::{
|
||||
fmt,
|
||||
future::Future,
|
||||
pin::Pin,
|
||||
sync::Arc,
|
||||
task::{Context, Poll},
|
||||
};
|
||||
use tokio::sync::{mpsc, oneshot};
|
||||
@ -41,7 +40,7 @@ where
|
||||
pub async fn resolve(
|
||||
&self,
|
||||
id: PayloadId,
|
||||
) -> Option<Result<Arc<BuiltPayload>, PayloadBuilderError>> {
|
||||
) -> Option<Result<Engine::BuiltPayload, PayloadBuilderError>> {
|
||||
self.inner.resolve(id).await
|
||||
}
|
||||
|
||||
@ -51,7 +50,7 @@ where
|
||||
pub async fn best_payload(
|
||||
&self,
|
||||
id: PayloadId,
|
||||
) -> Option<Result<Arc<BuiltPayload>, PayloadBuilderError>> {
|
||||
) -> Option<Result<Engine::BuiltPayload, PayloadBuilderError>> {
|
||||
self.inner.best_payload(id).await
|
||||
}
|
||||
|
||||
@ -81,7 +80,7 @@ where
|
||||
#[derive(Debug, Clone)]
|
||||
pub struct PayloadBuilderHandle<Engine: EngineTypes> {
|
||||
/// Sender half of the message channel to the [PayloadBuilderService].
|
||||
to_service: mpsc::UnboundedSender<PayloadServiceCommand<Engine::PayloadBuilderAttributes>>,
|
||||
to_service: mpsc::UnboundedSender<PayloadServiceCommand<Engine>>,
|
||||
}
|
||||
|
||||
// === impl PayloadBuilderHandle ===
|
||||
@ -94,9 +93,7 @@ where
|
||||
///
|
||||
/// Note: this is only used internally by the [PayloadBuilderService] to manage the payload
|
||||
/// building flow See [PayloadBuilderService::poll] for implementation details.
|
||||
pub fn new(
|
||||
to_service: mpsc::UnboundedSender<PayloadServiceCommand<Engine::PayloadBuilderAttributes>>,
|
||||
) -> Self {
|
||||
pub fn new(to_service: mpsc::UnboundedSender<PayloadServiceCommand<Engine>>) -> Self {
|
||||
Self { to_service }
|
||||
}
|
||||
|
||||
@ -107,7 +104,7 @@ where
|
||||
async fn resolve(
|
||||
&self,
|
||||
id: PayloadId,
|
||||
) -> Option<Result<Arc<BuiltPayload>, PayloadBuilderError>> {
|
||||
) -> Option<Result<Engine::BuiltPayload, PayloadBuilderError>> {
|
||||
let (tx, rx) = oneshot::channel();
|
||||
self.to_service.send(PayloadServiceCommand::Resolve(id, tx)).ok()?;
|
||||
match rx.await.transpose()? {
|
||||
@ -120,7 +117,7 @@ where
|
||||
async fn best_payload(
|
||||
&self,
|
||||
id: PayloadId,
|
||||
) -> Option<Result<Arc<BuiltPayload>, PayloadBuilderError>> {
|
||||
) -> Option<Result<Engine::BuiltPayload, PayloadBuilderError>> {
|
||||
let (tx, rx) = oneshot::channel();
|
||||
self.to_service.send(PayloadServiceCommand::BestPayload(id, tx)).ok()?;
|
||||
rx.await.ok()?
|
||||
@ -185,9 +182,9 @@ where
|
||||
/// All active payload jobs.
|
||||
payload_jobs: Vec<(Gen::Job, PayloadId)>,
|
||||
/// Copy of the sender half, so new [`PayloadBuilderHandle`] can be created on demand.
|
||||
service_tx: mpsc::UnboundedSender<PayloadServiceCommand<Engine::PayloadBuilderAttributes>>,
|
||||
service_tx: mpsc::UnboundedSender<PayloadServiceCommand<Engine>>,
|
||||
/// Receiver half of the command channel.
|
||||
command_rx: UnboundedReceiverStream<PayloadServiceCommand<Engine::PayloadBuilderAttributes>>,
|
||||
command_rx: UnboundedReceiverStream<PayloadServiceCommand<Engine>>,
|
||||
/// Metrics for the payload builder service
|
||||
metrics: PayloadBuilderServiceMetrics,
|
||||
/// Chain events notification stream
|
||||
@ -201,6 +198,7 @@ where
|
||||
Engine: EngineTypes,
|
||||
Gen: PayloadJobGenerator,
|
||||
Gen::Job: PayloadJob<PayloadAttributes = Engine::PayloadBuilderAttributes>,
|
||||
<Gen::Job as PayloadJob>::BuiltPayload: Into<Engine::BuiltPayload>,
|
||||
{
|
||||
/// Creates a new payload builder service and returns the [PayloadBuilderHandle] to interact
|
||||
/// with it.
|
||||
@ -236,14 +234,14 @@ where
|
||||
fn best_payload(
|
||||
&self,
|
||||
id: PayloadId,
|
||||
) -> Option<Result<Arc<BuiltPayload>, PayloadBuilderError>> {
|
||||
) -> Option<Result<Engine::BuiltPayload, PayloadBuilderError>> {
|
||||
let res = self
|
||||
.payload_jobs
|
||||
.iter()
|
||||
.find(|(_, job_id)| *job_id == id)
|
||||
.map(|(j, _)| j.best_payload());
|
||||
.map(|(j, _)| j.best_payload().map(|p| p.into()));
|
||||
if let Some(Ok(ref best)) = res {
|
||||
self.metrics.set_best_revenue(best.block.number, f64::from(best.fees));
|
||||
self.metrics.set_best_revenue(best.block().number, f64::from(best.fees()));
|
||||
}
|
||||
|
||||
res
|
||||
@ -251,7 +249,7 @@ where
|
||||
|
||||
/// Returns the best payload for the given identifier that has been built so far and terminates
|
||||
/// the job if requested.
|
||||
fn resolve(&mut self, id: PayloadId) -> Option<PayloadFuture> {
|
||||
fn resolve(&mut self, id: PayloadId) -> Option<PayloadFuture<Engine::BuiltPayload>> {
|
||||
trace!(%id, "resolving payload job");
|
||||
|
||||
let job = self.payload_jobs.iter().position(|(_, job_id)| *job_id == id)?;
|
||||
@ -269,9 +267,9 @@ where
|
||||
let res = fut.await;
|
||||
if let Ok(ref payload) = res {
|
||||
resolved_metrics
|
||||
.set_resolved_revenue(payload.block.number, f64::from(payload.fees));
|
||||
.set_resolved_revenue(payload.block().number, f64::from(payload.fees()));
|
||||
}
|
||||
res
|
||||
res.map(|p| p.into())
|
||||
};
|
||||
|
||||
Some(Box::pin(fut))
|
||||
@ -283,6 +281,7 @@ where
|
||||
Engine: EngineTypes,
|
||||
Gen: PayloadJobGenerator,
|
||||
Gen::Job: PayloadJob<PayloadAttributes = Engine::PayloadBuilderAttributes>,
|
||||
<Gen::Job as PayloadJob>::BuiltPayload: Into<Engine::BuiltPayload>,
|
||||
{
|
||||
/// Returns the payload attributes for the given payload.
|
||||
fn payload_attributes(
|
||||
@ -310,6 +309,7 @@ where
|
||||
<Gen as PayloadJobGenerator>::Job: Unpin + 'static,
|
||||
St: Stream<Item = CanonStateNotification> + Send + Unpin + 'static,
|
||||
Gen::Job: PayloadJob<PayloadAttributes = Engine::PayloadBuilderAttributes>,
|
||||
<Gen::Job as PayloadJob>::BuiltPayload: Into<Engine::BuiltPayload>,
|
||||
{
|
||||
type Output = ();
|
||||
|
||||
@ -398,24 +398,32 @@ where
|
||||
}
|
||||
|
||||
// TODO: make generic over built payload type
|
||||
type PayloadFuture =
|
||||
Pin<Box<dyn Future<Output = Result<Arc<BuiltPayload>, PayloadBuilderError>> + Send + Sync>>;
|
||||
type PayloadFuture<P> = Pin<Box<dyn Future<Output = Result<P, PayloadBuilderError>> + Send + Sync>>;
|
||||
|
||||
/// Message type for the [PayloadBuilderService].
|
||||
pub enum PayloadServiceCommand<T> {
|
||||
pub enum PayloadServiceCommand<Engine: EngineTypes> {
|
||||
/// Start building a new payload.
|
||||
BuildNewPayload(T, oneshot::Sender<Result<PayloadId, PayloadBuilderError>>),
|
||||
BuildNewPayload(
|
||||
Engine::PayloadBuilderAttributes,
|
||||
oneshot::Sender<Result<PayloadId, PayloadBuilderError>>,
|
||||
),
|
||||
/// Get the best payload so far
|
||||
BestPayload(PayloadId, oneshot::Sender<Option<Result<Arc<BuiltPayload>, PayloadBuilderError>>>),
|
||||
BestPayload(
|
||||
PayloadId,
|
||||
oneshot::Sender<Option<Result<Engine::BuiltPayload, PayloadBuilderError>>>,
|
||||
),
|
||||
/// Get the payload attributes for the given payload
|
||||
PayloadAttributes(PayloadId, oneshot::Sender<Option<Result<T, PayloadBuilderError>>>),
|
||||
PayloadAttributes(
|
||||
PayloadId,
|
||||
oneshot::Sender<Option<Result<Engine::PayloadBuilderAttributes, PayloadBuilderError>>>,
|
||||
),
|
||||
/// Resolve the payload and return the payload
|
||||
Resolve(PayloadId, oneshot::Sender<Option<PayloadFuture>>),
|
||||
Resolve(PayloadId, oneshot::Sender<Option<PayloadFuture<Engine::BuiltPayload>>>),
|
||||
}
|
||||
|
||||
impl<T> fmt::Debug for PayloadServiceCommand<T>
|
||||
impl<Engine> fmt::Debug for PayloadServiceCommand<Engine>
|
||||
where
|
||||
T: fmt::Debug,
|
||||
Engine: EngineTypes,
|
||||
{
|
||||
fn fmt(&self, f: &mut fmt::Formatter<'_>) -> fmt::Result {
|
||||
match self {
|
||||
|
||||
@ -1,7 +1,7 @@
|
||||
//! Utils for testing purposes.
|
||||
|
||||
use crate::{
|
||||
error::PayloadBuilderError, traits::KeepPayloadJobAlive, BuiltPayload,
|
||||
error::PayloadBuilderError, traits::KeepPayloadJobAlive, EthBuiltPayload,
|
||||
EthPayloadBuilderAttributes, PayloadBuilderHandle, PayloadBuilderService, PayloadJob,
|
||||
PayloadJobGenerator,
|
||||
};
|
||||
@ -11,7 +11,6 @@ use reth_provider::CanonStateNotification;
|
||||
use std::{
|
||||
future::Future,
|
||||
pin::Pin,
|
||||
sync::Arc,
|
||||
task::{Context, Poll},
|
||||
};
|
||||
|
||||
@ -25,7 +24,10 @@ pub fn test_payload_service<Engine>() -> (
|
||||
PayloadBuilderHandle<Engine>,
|
||||
)
|
||||
where
|
||||
Engine: EngineTypes<PayloadBuilderAttributes = EthPayloadBuilderAttributes>,
|
||||
Engine: EngineTypes<
|
||||
PayloadBuilderAttributes = EthPayloadBuilderAttributes,
|
||||
BuiltPayload = EthBuiltPayload,
|
||||
>,
|
||||
{
|
||||
PayloadBuilderService::new(Default::default(), futures_util::stream::empty())
|
||||
}
|
||||
@ -33,7 +35,10 @@ where
|
||||
/// Creates a new [PayloadBuilderService] for testing purposes and spawns it in the background.
|
||||
pub fn spawn_test_payload_service<Engine>() -> PayloadBuilderHandle<Engine>
|
||||
where
|
||||
Engine: EngineTypes<PayloadBuilderAttributes = EthPayloadBuilderAttributes> + 'static,
|
||||
Engine: EngineTypes<
|
||||
PayloadBuilderAttributes = EthPayloadBuilderAttributes,
|
||||
BuiltPayload = EthBuiltPayload,
|
||||
> + 'static,
|
||||
{
|
||||
let (service, handle) = test_payload_service();
|
||||
tokio::spawn(service);
|
||||
@ -73,14 +78,11 @@ impl Future for TestPayloadJob {
|
||||
impl PayloadJob for TestPayloadJob {
|
||||
type PayloadAttributes = EthPayloadBuilderAttributes;
|
||||
type ResolvePayloadFuture =
|
||||
futures_util::future::Ready<Result<Arc<BuiltPayload>, PayloadBuilderError>>;
|
||||
futures_util::future::Ready<Result<EthBuiltPayload, PayloadBuilderError>>;
|
||||
type BuiltPayload = EthBuiltPayload;
|
||||
|
||||
fn best_payload(&self) -> Result<Arc<BuiltPayload>, PayloadBuilderError> {
|
||||
Ok(Arc::new(BuiltPayload::new(
|
||||
self.attr.payload_id(),
|
||||
Block::default().seal_slow(),
|
||||
U256::ZERO,
|
||||
)))
|
||||
fn best_payload(&self) -> Result<EthBuiltPayload, PayloadBuilderError> {
|
||||
Ok(EthBuiltPayload::new(self.attr.payload_id(), Block::default().seal_slow(), U256::ZERO))
|
||||
}
|
||||
|
||||
fn payload_attributes(&self) -> Result<EthPayloadBuilderAttributes, PayloadBuilderError> {
|
||||
|
||||
@ -1,9 +1,9 @@
|
||||
//! Trait abstractions used by the payload crate.
|
||||
|
||||
use crate::{error::PayloadBuilderError, BuiltPayload};
|
||||
use reth_node_api::PayloadBuilderAttributes;
|
||||
use crate::error::PayloadBuilderError;
|
||||
use reth_node_api::{BuiltPayload, PayloadBuilderAttributes};
|
||||
use reth_provider::CanonStateNotification;
|
||||
use std::{future::Future, sync::Arc};
|
||||
use std::future::Future;
|
||||
|
||||
/// A type that can build a payload.
|
||||
///
|
||||
@ -20,15 +20,17 @@ pub trait PayloadJob: Future<Output = Result<(), PayloadBuilderError>> + Send +
|
||||
/// Represents the payload attributes type that is used to spawn this payload job.
|
||||
type PayloadAttributes: PayloadBuilderAttributes + std::fmt::Debug;
|
||||
/// Represents the future that resolves the block that's returned to the CL.
|
||||
type ResolvePayloadFuture: Future<Output = Result<Arc<BuiltPayload>, PayloadBuilderError>>
|
||||
type ResolvePayloadFuture: Future<Output = Result<Self::BuiltPayload, PayloadBuilderError>>
|
||||
+ Send
|
||||
+ Sync
|
||||
+ 'static;
|
||||
/// Represents the built payload type that is returned to the CL.
|
||||
type BuiltPayload: BuiltPayload + std::fmt::Debug;
|
||||
|
||||
/// Returns the best payload that has been built so far.
|
||||
///
|
||||
/// Note: This is never called by the CL.
|
||||
fn best_payload(&self) -> Result<Arc<BuiltPayload>, PayloadBuilderError>;
|
||||
fn best_payload(&self) -> Result<Self::BuiltPayload, PayloadBuilderError>;
|
||||
|
||||
/// Returns the payload attributes for the payload being built.
|
||||
fn payload_attributes(&self) -> Result<Self::PayloadAttributes, PayloadBuilderError>;
|
||||
|
||||
@ -17,7 +17,7 @@ mod builder {
|
||||
BuildOutcome, PayloadBuilder, PayloadConfig, WithdrawalsOutcome,
|
||||
};
|
||||
use reth_payload_builder::{
|
||||
error::PayloadBuilderError, BuiltPayload, EthPayloadBuilderAttributes,
|
||||
error::PayloadBuilderError, EthBuiltPayload, EthPayloadBuilderAttributes,
|
||||
};
|
||||
use reth_primitives::{
|
||||
constants::{eip4844::MAX_DATA_GAS_PER_BLOCK, BEACON_NONCE},
|
||||
@ -48,11 +48,12 @@ mod builder {
|
||||
Pool: TransactionPool,
|
||||
{
|
||||
type Attributes = EthPayloadBuilderAttributes;
|
||||
type BuiltPayload = EthBuiltPayload;
|
||||
|
||||
fn try_build(
|
||||
&self,
|
||||
args: BuildArguments<Pool, Client, EthPayloadBuilderAttributes>,
|
||||
) -> Result<BuildOutcome, PayloadBuilderError> {
|
||||
args: BuildArguments<Pool, Client, EthPayloadBuilderAttributes, EthBuiltPayload>,
|
||||
) -> Result<BuildOutcome<EthBuiltPayload>, PayloadBuilderError> {
|
||||
default_ethereum_payload_builder(args)
|
||||
}
|
||||
}
|
||||
@ -64,8 +65,8 @@ mod builder {
|
||||
/// a result indicating success with the payload or an error in case of failure.
|
||||
#[inline]
|
||||
pub fn default_ethereum_payload_builder<Pool, Client>(
|
||||
args: BuildArguments<Pool, Client, EthPayloadBuilderAttributes>,
|
||||
) -> Result<BuildOutcome, PayloadBuilderError>
|
||||
args: BuildArguments<Pool, Client, EthPayloadBuilderAttributes, EthBuiltPayload>,
|
||||
) -> Result<BuildOutcome<EthBuiltPayload>, PayloadBuilderError>
|
||||
where
|
||||
Client: StateProviderFactory,
|
||||
Pool: TransactionPool,
|
||||
@ -218,7 +219,7 @@ mod builder {
|
||||
}
|
||||
|
||||
// check if we have a better block
|
||||
if !is_better_payload(best_payload.as_deref(), total_fees) {
|
||||
if !is_better_payload(best_payload.as_ref(), total_fees) {
|
||||
// can skip building the block
|
||||
return Ok(BuildOutcome::Aborted { fees: total_fees, cached_reads })
|
||||
}
|
||||
@ -298,7 +299,7 @@ mod builder {
|
||||
let sealed_block = block.seal_slow();
|
||||
debug!(target: "payload_builder", ?sealed_block, "sealed built block");
|
||||
|
||||
let mut payload = BuiltPayload::new(attributes.id, sealed_block, total_fees);
|
||||
let mut payload = EthBuiltPayload::new(attributes.id, sealed_block, total_fees);
|
||||
|
||||
// extend the payload with the blob sidecars from the executed txs
|
||||
payload.extend_sidecars(blob_sidecars);
|
||||
|
||||
@ -18,7 +18,7 @@ mod builder {
|
||||
use reth_basic_payload_builder::*;
|
||||
use reth_node_api::PayloadBuilderAttributes;
|
||||
use reth_payload_builder::{
|
||||
error::PayloadBuilderError, BuiltPayload, OptimismPayloadBuilderAttributes,
|
||||
error::PayloadBuilderError, EthBuiltPayload, OptimismPayloadBuilderAttributes,
|
||||
};
|
||||
use reth_primitives::{
|
||||
constants::BEACON_NONCE,
|
||||
@ -35,7 +35,6 @@ mod builder {
|
||||
primitives::{EVMError, Env, InvalidTransaction, ResultAndState},
|
||||
DatabaseCommit, State,
|
||||
};
|
||||
use std::sync::Arc;
|
||||
use tracing::{debug, trace};
|
||||
|
||||
/// Optimism's payload builder
|
||||
@ -72,18 +71,19 @@ mod builder {
|
||||
Pool: TransactionPool,
|
||||
{
|
||||
type Attributes = OptimismPayloadBuilderAttributes;
|
||||
type BuiltPayload = EthBuiltPayload;
|
||||
|
||||
fn try_build(
|
||||
&self,
|
||||
args: BuildArguments<Pool, Client, OptimismPayloadBuilderAttributes>,
|
||||
) -> Result<BuildOutcome, PayloadBuilderError> {
|
||||
args: BuildArguments<Pool, Client, OptimismPayloadBuilderAttributes, EthBuiltPayload>,
|
||||
) -> Result<BuildOutcome<EthBuiltPayload>, PayloadBuilderError> {
|
||||
optimism_payload_builder(args, self.compute_pending_block)
|
||||
}
|
||||
|
||||
fn on_missing_payload(
|
||||
&self,
|
||||
args: BuildArguments<Pool, Client, OptimismPayloadBuilderAttributes>,
|
||||
) -> Option<Arc<BuiltPayload>> {
|
||||
args: BuildArguments<Pool, Client, OptimismPayloadBuilderAttributes, EthBuiltPayload>,
|
||||
) -> Option<EthBuiltPayload> {
|
||||
// In Optimism, the PayloadAttributes can specify a `no_tx_pool` option that implies we
|
||||
// should not pull transactions from the tx pool. In this case, we build the payload
|
||||
// upfront with the list of transactions sent in the attributes without caring about
|
||||
@ -91,7 +91,6 @@ mod builder {
|
||||
if args.config.attributes.no_tx_pool {
|
||||
if let Ok(BuildOutcome::Better { payload, .. }) = self.try_build(args) {
|
||||
trace!(target: "payload_builder", "[OPTIMISM] Forced best payload");
|
||||
let payload = Arc::new(payload);
|
||||
return Some(payload)
|
||||
}
|
||||
}
|
||||
@ -110,9 +109,9 @@ mod builder {
|
||||
/// a result indicating success with the payload or an error in case of failure.
|
||||
#[inline]
|
||||
pub(crate) fn optimism_payload_builder<Pool, Client>(
|
||||
args: BuildArguments<Pool, Client, OptimismPayloadBuilderAttributes>,
|
||||
args: BuildArguments<Pool, Client, OptimismPayloadBuilderAttributes, EthBuiltPayload>,
|
||||
_compute_pending_block: bool,
|
||||
) -> Result<BuildOutcome, PayloadBuilderError>
|
||||
) -> Result<BuildOutcome<EthBuiltPayload>, PayloadBuilderError>
|
||||
where
|
||||
Client: StateProviderFactory,
|
||||
Pool: TransactionPool,
|
||||
@ -340,7 +339,7 @@ mod builder {
|
||||
}
|
||||
|
||||
// check if we have a better block
|
||||
if !is_better_payload(best_payload.as_deref(), total_fees) {
|
||||
if !is_better_payload(best_payload.as_ref(), total_fees) {
|
||||
// can skip building the block
|
||||
return Ok(BuildOutcome::Aborted { fees: total_fees, cached_reads })
|
||||
}
|
||||
@ -406,7 +405,7 @@ mod builder {
|
||||
let sealed_block = block.seal_slow();
|
||||
debug!(target: "payload_builder", ?sealed_block, "sealed built block");
|
||||
|
||||
let mut payload = BuiltPayload::new(attributes.payload_id(), sealed_block, total_fees);
|
||||
let mut payload = EthBuiltPayload::new(attributes.payload_id(), sealed_block, total_fees);
|
||||
|
||||
// extend the payload with the blob sidecars from the executed txs
|
||||
payload.extend_sidecars(blob_sidecars);
|
||||
|
||||
@ -4,8 +4,9 @@ use jsonrpsee_core::RpcResult;
|
||||
use reth_beacon_consensus::BeaconConsensusEngineHandle;
|
||||
use reth_interfaces::consensus::ForkchoiceState;
|
||||
use reth_node_api::{
|
||||
validate_payload_timestamp, validate_version_specific_fields, EngineApiMessageVersion,
|
||||
EngineTypes, PayloadAttributes, PayloadBuilderAttributes, PayloadOrAttributes,
|
||||
validate_payload_timestamp, validate_version_specific_fields, BuiltPayload,
|
||||
EngineApiMessageVersion, EngineTypes, PayloadAttributes, PayloadBuilderAttributes,
|
||||
PayloadOrAttributes,
|
||||
};
|
||||
use reth_payload_builder::PayloadStore;
|
||||
use reth_primitives::{BlockHash, BlockHashOrNumber, BlockNumber, ChainSpec, Hardfork, B256, U64};
|
||||
@ -55,7 +56,6 @@ impl<Provider, EngineT> EngineApi<Provider, EngineT>
|
||||
where
|
||||
Provider: HeaderProvider + BlockReader + StateProviderFactory + EvmEnvProvider + 'static,
|
||||
EngineT: EngineTypes + 'static,
|
||||
EngineT::PayloadBuilderAttributes: Send,
|
||||
{
|
||||
/// Create new instance of [EngineApi].
|
||||
pub fn new(
|
||||
@ -210,7 +210,7 @@ where
|
||||
.resolve(payload_id)
|
||||
.await
|
||||
.ok_or(EngineApiError::UnknownPayload)?
|
||||
.map(|payload| (*payload).clone().into_v1_payload())?)
|
||||
.map(|payload| payload.into_v1_payload())?)
|
||||
}
|
||||
|
||||
/// Returns the most recent version of the payload that is available in the corresponding
|
||||
@ -241,7 +241,7 @@ where
|
||||
.resolve(payload_id)
|
||||
.await
|
||||
.ok_or(EngineApiError::UnknownPayload)?
|
||||
.map(|payload| (*payload).clone().into_v2_payload())?)
|
||||
.map(|payload| payload.into_v2_payload())?)
|
||||
}
|
||||
|
||||
/// Returns the most recent version of the payload that is available in the corresponding
|
||||
@ -272,7 +272,7 @@ where
|
||||
.resolve(payload_id)
|
||||
.await
|
||||
.ok_or(EngineApiError::UnknownPayload)?
|
||||
.map(|payload| (*payload).clone().into_v3_payload())?)
|
||||
.map(|payload| payload.into_v3_payload())?)
|
||||
}
|
||||
|
||||
/// Returns the execution payload bodies by the range starting at `start`, containing `count`
|
||||
@ -471,9 +471,7 @@ where
|
||||
impl<Provider, EngineT> EngineApiServer<EngineT> for EngineApi<Provider, EngineT>
|
||||
where
|
||||
Provider: HeaderProvider + BlockReader + StateProviderFactory + EvmEnvProvider + 'static,
|
||||
EngineT: EngineTypes + 'static + Send,
|
||||
EngineT::PayloadAttributes: Send,
|
||||
EngineT::PayloadBuilderAttributes: Send,
|
||||
EngineT: EngineTypes + 'static,
|
||||
{
|
||||
/// Handler for `engine_newPayloadV1`
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/3d627c95a4d3510a8187dd02e0250ecb4331d27e/src/engine/paris.md#engine_newpayloadv1>
|
||||
|
||||
Reference in New Issue
Block a user