mirror of
https://github.com/paradigmxyz/reth.git
synced 2026-01-29 09:08:05 -05:00
feat(rpc): basic engine api (#551)
* feat(rpc): engine api * change transition config exchange * payload block construction * pull out engine api logic * linter * clippy * clippy * Apply suggestions from code review Co-authored-by: Matthias Seitz <matthias.seitz@outlook.de> * import & map_err for RecvError * move result Co-authored-by: Matthias Seitz <matthias.seitz@outlook.de>
This commit is contained in:
5
Cargo.lock
generated
5
Cargo.lock
generated
@@ -3336,13 +3336,16 @@ dependencies = [
|
||||
"async-trait",
|
||||
"auto_impl",
|
||||
"eyre",
|
||||
"futures",
|
||||
"reth-interfaces",
|
||||
"reth-primitives",
|
||||
"reth-provider",
|
||||
"reth-rlp",
|
||||
"reth-rpc-types",
|
||||
"serde",
|
||||
"thiserror",
|
||||
"tokio",
|
||||
"tokio-stream",
|
||||
]
|
||||
|
||||
[[package]]
|
||||
@@ -3721,6 +3724,7 @@ dependencies = [
|
||||
"async-trait",
|
||||
"hex",
|
||||
"jsonrpsee",
|
||||
"reth-consensus",
|
||||
"reth-interfaces",
|
||||
"reth-network",
|
||||
"reth-primitives",
|
||||
@@ -3731,6 +3735,7 @@ dependencies = [
|
||||
"serde",
|
||||
"serde_json",
|
||||
"thiserror",
|
||||
"tokio",
|
||||
]
|
||||
|
||||
[[package]]
|
||||
|
||||
@@ -11,14 +11,19 @@ readme = "README.md"
|
||||
reth-primitives = { path = "../primitives" }
|
||||
reth-interfaces = { path = "../interfaces" }
|
||||
reth-provider = { path = "../storage/provider" }
|
||||
reth-rlp = {path = "../common/rlp"}
|
||||
reth-rlp = { path = "../common/rlp" }
|
||||
reth-rpc-types = { path = "../net/rpc-types" }
|
||||
|
||||
# async
|
||||
futures = "0.3"
|
||||
async-trait = "0.1.57"
|
||||
tokio = { version = "1", features = ["sync"] }
|
||||
tokio-stream = "0.1"
|
||||
|
||||
# common
|
||||
async-trait = "0.1.57"
|
||||
thiserror = "1.0.37"
|
||||
eyre = "0.6.8"
|
||||
auto_impl = "1.0"
|
||||
tokio = { version = "1.21.2", features = ["sync"] }
|
||||
|
||||
# io
|
||||
serde = { version = "1.0", optional = true }
|
||||
|
||||
67
crates/consensus/src/engine/error.rs
Normal file
67
crates/consensus/src/engine/error.rs
Normal file
@@ -0,0 +1,67 @@
|
||||
use reth_primitives::{Bytes, H256, U256};
|
||||
use thiserror::Error;
|
||||
|
||||
/// The Engine API result type
|
||||
pub type EngineApiResult<Ok> = Result<Ok, EngineApiError>;
|
||||
|
||||
/// Error returned by [`EngineApi`][crate::engine::EngineApi]
|
||||
#[derive(Error, Debug)]
|
||||
pub enum EngineApiError {
|
||||
/// Invalid payload extra data.
|
||||
#[error("Invalid payload extra data: {0}")]
|
||||
PayloadExtraData(Bytes),
|
||||
/// Invalid payload base fee.
|
||||
#[error("Invalid payload base fee: {0}")]
|
||||
PayloadBaseFee(U256),
|
||||
/// Invalid payload block hash.
|
||||
#[error("Invalid payload block hash. Execution: {execution}. Consensus: {consensus}")]
|
||||
PayloadBlockHash {
|
||||
/// The block hash computed from the payload.
|
||||
execution: H256,
|
||||
/// The block hash provided with the payload.
|
||||
consensus: H256,
|
||||
},
|
||||
/// Invalid payload block hash.
|
||||
#[error("Invalid payload timestamp: {invalid}. Latest: {latest}")]
|
||||
PayloadTimestamp {
|
||||
/// The payload timestamp.
|
||||
invalid: u64,
|
||||
/// Latest available timestamp.
|
||||
latest: u64,
|
||||
},
|
||||
/// Received pre-merge payload.
|
||||
#[error("Received pre-merge payload.")]
|
||||
PayloadPreMerge,
|
||||
/// Unknown payload requested.
|
||||
#[error("Unknown payload")]
|
||||
PayloadUnknown,
|
||||
/// Terminal total difficulty mismatch during transition configuration exchange.
|
||||
#[error(
|
||||
"Invalid transition terminal total difficulty. Execution: {execution}. Consensus: {consensus}"
|
||||
)]
|
||||
TerminalTD {
|
||||
/// Execution terminal total difficulty value.
|
||||
execution: U256,
|
||||
/// Consensus terminal total difficulty value.
|
||||
consensus: U256,
|
||||
},
|
||||
/// Terminal block hash mismatch during transition configuration exchange.
|
||||
#[error(
|
||||
"Invalid transition terminal block hash. Execution: {execution:?}. Consensus: {consensus}"
|
||||
)]
|
||||
TerminalBlockHash {
|
||||
/// Execution terminal block hash. `None` if block number is not found in the database.
|
||||
execution: Option<H256>,
|
||||
/// Consensus terminal block hash.
|
||||
consensus: H256,
|
||||
},
|
||||
/// Forkchoice zero hash head received.
|
||||
#[error("Received zero hash as forkchoice head")]
|
||||
ForkchoiceEmptyHead,
|
||||
/// Encountered decoding error.
|
||||
#[error(transparent)]
|
||||
Decode(#[from] reth_rlp::DecodeError),
|
||||
/// API encountered an internal error.
|
||||
#[error(transparent)]
|
||||
Internal(#[from] reth_interfaces::Error),
|
||||
}
|
||||
293
crates/consensus/src/engine/mod.rs
Normal file
293
crates/consensus/src/engine/mod.rs
Normal file
@@ -0,0 +1,293 @@
|
||||
use futures::StreamExt;
|
||||
use reth_interfaces::consensus::ForkchoiceState;
|
||||
use reth_primitives::{
|
||||
proofs::{self, EMPTY_LIST_HASH},
|
||||
rpc::BlockId,
|
||||
BlockLocked, Header, TransactionSigned, H64,
|
||||
};
|
||||
use reth_provider::{BlockProvider, HeaderProvider};
|
||||
use reth_rlp::Decodable;
|
||||
use reth_rpc_types::engine::{
|
||||
ExecutionPayload, ForkchoiceUpdated, PayloadAttributes, PayloadStatus, PayloadStatusEnum,
|
||||
TransitionConfiguration,
|
||||
};
|
||||
use std::{
|
||||
collections::HashMap,
|
||||
future::Future,
|
||||
pin::Pin,
|
||||
sync::Arc,
|
||||
task::{ready, Context, Poll},
|
||||
};
|
||||
use tokio::sync::oneshot;
|
||||
use tokio_stream::wrappers::UnboundedReceiverStream;
|
||||
|
||||
mod error;
|
||||
use crate::Config;
|
||||
pub use error::{EngineApiError, EngineApiResult};
|
||||
|
||||
/// The Engine API response sender
|
||||
pub type EngineApiSender<Ok> = oneshot::Sender<EngineApiResult<Ok>>;
|
||||
|
||||
/// Consensus engine API trait.
|
||||
pub trait ConsensusEngine {
|
||||
/// Retrieves payload from local cache.
|
||||
fn get_payload(&self, payload_id: H64) -> Option<ExecutionPayload>;
|
||||
|
||||
/// Receives a payload to validate and execute.
|
||||
fn new_payload(&mut self, payload: ExecutionPayload) -> EngineApiResult<PayloadStatus>;
|
||||
|
||||
/// Updates the fork choice state
|
||||
fn fork_choice_updated(
|
||||
&self,
|
||||
fork_choice_state: ForkchoiceState,
|
||||
payload_attributes: Option<PayloadAttributes>,
|
||||
) -> EngineApiResult<ForkchoiceUpdated>;
|
||||
|
||||
/// Verifies the transition configuration between execution and consensus clients.
|
||||
fn exchange_transition_configuration(
|
||||
&self,
|
||||
config: TransitionConfiguration,
|
||||
) -> EngineApiResult<TransitionConfiguration>;
|
||||
}
|
||||
|
||||
/// Message type for communicating with [EthConsensusEngine]
|
||||
pub enum EngineMessage {
|
||||
/// New payload message
|
||||
NewPayload(ExecutionPayload, EngineApiSender<PayloadStatus>),
|
||||
/// Get payload message
|
||||
GetPayload(H64, EngineApiSender<ExecutionPayload>),
|
||||
/// Forkchoice updated message
|
||||
ForkchoiceUpdated(
|
||||
ForkchoiceState,
|
||||
Option<PayloadAttributes>,
|
||||
EngineApiSender<ForkchoiceUpdated>,
|
||||
),
|
||||
/// Exchange transition configuration message
|
||||
ExchangeTransitionConfiguration(
|
||||
TransitionConfiguration,
|
||||
EngineApiSender<TransitionConfiguration>,
|
||||
),
|
||||
}
|
||||
|
||||
/// The consensus engine API implementation
|
||||
#[must_use = "EthConsensusEngine does nothing unless polled."]
|
||||
pub struct EthConsensusEngine<Client> {
|
||||
/// Consensus configuration
|
||||
config: Config,
|
||||
client: Arc<Client>,
|
||||
/// Placeholder for storing future blocks
|
||||
local_store: HashMap<H64, ExecutionPayload>, // TODO: bound
|
||||
// remote_store: HashMap<H64, ExecutionPayload>,
|
||||
rx: UnboundedReceiverStream<EngineMessage>,
|
||||
}
|
||||
|
||||
impl<Client: HeaderProvider + BlockProvider> EthConsensusEngine<Client> {
|
||||
fn on_message(&mut self, msg: EngineMessage) {
|
||||
match msg {
|
||||
EngineMessage::GetPayload(payload_id, tx) => {
|
||||
// NOTE: Will always result in `PayloadUnknown` since we don't support block
|
||||
// building for now.
|
||||
let _ = tx.send(self.get_payload(payload_id).ok_or(EngineApiError::PayloadUnknown));
|
||||
}
|
||||
EngineMessage::NewPayload(payload, tx) => {
|
||||
let _ = tx.send(self.new_payload(payload));
|
||||
}
|
||||
EngineMessage::ForkchoiceUpdated(state, attrs, tx) => {
|
||||
let _ = tx.send(self.fork_choice_updated(state, attrs));
|
||||
}
|
||||
EngineMessage::ExchangeTransitionConfiguration(config, tx) => {
|
||||
let _ = tx.send(self.exchange_transition_configuration(config));
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
/// Try to construct a block from given payload. Perform addition validation of `extra_data` and
|
||||
/// `base_fee_per_gas` fields.
|
||||
///
|
||||
/// NOTE: The log bloom is assumed to be validated during serialization.
|
||||
/// NOTE: Ommers hash is validated upon computing block hash and comparing the value with
|
||||
/// `payload.block_hash`.
|
||||
/// Ref: https://github.com/ethereum/go-ethereum/blob/79a478bb6176425c2400e949890e668a3d9a3d05/core/beacon/types.go#L145
|
||||
fn try_construct_block(&self, payload: ExecutionPayload) -> EngineApiResult<BlockLocked> {
|
||||
if payload.extra_data.len() > 32 {
|
||||
return Err(EngineApiError::PayloadExtraData(payload.extra_data))
|
||||
}
|
||||
|
||||
if payload.base_fee_per_gas.is_zero() {
|
||||
return Err(EngineApiError::PayloadBaseFee(payload.base_fee_per_gas))
|
||||
}
|
||||
|
||||
let transactions = payload
|
||||
.transactions
|
||||
.iter()
|
||||
.map(|tx| TransactionSigned::decode(&mut tx.as_ref()))
|
||||
.collect::<std::result::Result<Vec<_>, _>>()?;
|
||||
let transactions_root = proofs::calculate_transaction_root(transactions.iter());
|
||||
let header = Header {
|
||||
parent_hash: payload.parent_hash,
|
||||
beneficiary: payload.fee_recipient,
|
||||
state_root: payload.state_root,
|
||||
transactions_root,
|
||||
receipts_root: payload.receipts_root,
|
||||
logs_bloom: payload.logs_bloom,
|
||||
number: payload.block_number.as_u64(),
|
||||
gas_limit: payload.gas_limit.as_u64(),
|
||||
gas_used: payload.gas_used.as_u64(),
|
||||
timestamp: payload.timestamp.as_u64(),
|
||||
mix_hash: payload.prev_randao,
|
||||
base_fee_per_gas: Some(payload.base_fee_per_gas.as_u64()),
|
||||
extra_data: payload.extra_data.0,
|
||||
// Defaults
|
||||
ommers_hash: EMPTY_LIST_HASH,
|
||||
difficulty: Default::default(),
|
||||
nonce: Default::default(),
|
||||
};
|
||||
let header = header.seal();
|
||||
|
||||
if payload.block_hash != header.hash() {
|
||||
return Err(EngineApiError::PayloadBlockHash {
|
||||
execution: header.hash(),
|
||||
consensus: payload.block_hash,
|
||||
})
|
||||
}
|
||||
|
||||
Ok(BlockLocked { header, body: transactions, ommers: Default::default() })
|
||||
}
|
||||
}
|
||||
|
||||
impl<Client: HeaderProvider + BlockProvider> ConsensusEngine for EthConsensusEngine<Client> {
|
||||
fn get_payload(&self, payload_id: H64) -> Option<ExecutionPayload> {
|
||||
self.local_store.get(&payload_id).cloned()
|
||||
}
|
||||
|
||||
fn new_payload(&mut self, payload: ExecutionPayload) -> EngineApiResult<PayloadStatus> {
|
||||
let block = match self.try_construct_block(payload) {
|
||||
Ok(b) => b,
|
||||
Err(err) => {
|
||||
return Ok(PayloadStatus::from_status(PayloadStatusEnum::InvalidBlockHash {
|
||||
validation_error: err.to_string(),
|
||||
}))
|
||||
}
|
||||
};
|
||||
|
||||
// The block already exists in our database
|
||||
if self.client.is_known(&block.hash())? {
|
||||
return Ok(PayloadStatus::new(PayloadStatusEnum::Valid, block.hash()))
|
||||
}
|
||||
|
||||
let Some(parent) = self.client.block(BlockId::Hash(block.parent_hash))? else {
|
||||
// TODO: cache block for storing later
|
||||
return Ok(PayloadStatus::from_status(PayloadStatusEnum::Syncing))
|
||||
};
|
||||
|
||||
let parent_td = self.client.header_td(&block.parent_hash)?;
|
||||
if parent_td.unwrap_or_default() <= self.config.merge_terminal_total_difficulty.into() {
|
||||
return Ok(PayloadStatus::from_status(PayloadStatusEnum::Invalid {
|
||||
validation_error: EngineApiError::PayloadPreMerge.to_string(),
|
||||
}))
|
||||
}
|
||||
|
||||
if block.timestamp <= parent.timestamp {
|
||||
return Err(EngineApiError::PayloadTimestamp {
|
||||
invalid: block.timestamp,
|
||||
latest: parent.timestamp,
|
||||
})
|
||||
}
|
||||
|
||||
// TODO: execute block
|
||||
|
||||
Ok(PayloadStatus::new(PayloadStatusEnum::Valid, block.hash()))
|
||||
}
|
||||
|
||||
fn fork_choice_updated(
|
||||
&self,
|
||||
fork_choice_state: ForkchoiceState,
|
||||
_payload_attributes: Option<PayloadAttributes>,
|
||||
) -> EngineApiResult<ForkchoiceUpdated> {
|
||||
let ForkchoiceState { head_block_hash, finalized_block_hash, .. } = fork_choice_state;
|
||||
|
||||
if head_block_hash.is_zero() {
|
||||
return Ok(ForkchoiceUpdated::from_status(PayloadStatusEnum::Invalid {
|
||||
validation_error: EngineApiError::ForkchoiceEmptyHead.to_string(),
|
||||
}))
|
||||
}
|
||||
|
||||
// Block is not known, nothing to do.
|
||||
if !self.client.is_known(&head_block_hash)? {
|
||||
return Ok(ForkchoiceUpdated::from_status(PayloadStatusEnum::Syncing))
|
||||
}
|
||||
|
||||
// The finalized block hash is not known, we are still syncing
|
||||
if !finalized_block_hash.is_zero() && !self.client.is_known(&finalized_block_hash)? {
|
||||
return Ok(ForkchoiceUpdated::from_status(PayloadStatusEnum::Syncing))
|
||||
}
|
||||
|
||||
let chain_info = self.client.chain_info()?;
|
||||
Ok(ForkchoiceUpdated::from_status(PayloadStatusEnum::Valid)
|
||||
.with_latest_valid_hash(chain_info.best_hash))
|
||||
}
|
||||
|
||||
fn exchange_transition_configuration(
|
||||
&self,
|
||||
config: TransitionConfiguration,
|
||||
) -> EngineApiResult<TransitionConfiguration> {
|
||||
let TransitionConfiguration {
|
||||
terminal_total_difficulty,
|
||||
terminal_block_hash,
|
||||
terminal_block_number,
|
||||
} = config;
|
||||
|
||||
// Compare total difficulty values
|
||||
let merge_terminal_td = self.config.merge_terminal_total_difficulty.into();
|
||||
if merge_terminal_td != terminal_total_difficulty {
|
||||
return Err(EngineApiError::TerminalTD {
|
||||
execution: merge_terminal_td,
|
||||
consensus: terminal_total_difficulty,
|
||||
})
|
||||
}
|
||||
|
||||
// Short circuit if communicated block hash is zero
|
||||
if terminal_block_hash.is_zero() {
|
||||
return Ok(TransitionConfiguration {
|
||||
terminal_total_difficulty: merge_terminal_td,
|
||||
..Default::default()
|
||||
})
|
||||
}
|
||||
|
||||
// Attempt to look up terminal block hash
|
||||
let local_hash = self.client.block_hash(terminal_block_number.into())?;
|
||||
|
||||
// Transition configuration exchange is successful if block hashes match
|
||||
match local_hash {
|
||||
Some(hash) if hash == terminal_block_hash => Ok(TransitionConfiguration {
|
||||
terminal_total_difficulty: merge_terminal_td,
|
||||
terminal_block_hash,
|
||||
terminal_block_number,
|
||||
}),
|
||||
_ => Err(EngineApiError::TerminalBlockHash {
|
||||
execution: local_hash,
|
||||
consensus: terminal_block_hash,
|
||||
}),
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
impl<Client> Future for EthConsensusEngine<Client>
|
||||
where
|
||||
Client: HeaderProvider + BlockProvider + Unpin,
|
||||
{
|
||||
type Output = ();
|
||||
|
||||
fn poll(self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Self::Output> {
|
||||
let this = self.get_mut();
|
||||
loop {
|
||||
match ready!(this.rx.poll_next_unpin(cx)) {
|
||||
Some(msg) => this.on_message(msg),
|
||||
None => {
|
||||
// channel closed
|
||||
return Poll::Ready(())
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
@@ -13,6 +13,9 @@ pub mod config;
|
||||
pub mod consensus;
|
||||
pub mod verification;
|
||||
|
||||
/// Engine API module.
|
||||
pub mod engine;
|
||||
|
||||
pub use config::Config;
|
||||
pub use consensus::BeaconConsensus;
|
||||
pub use reth_interfaces::consensus::Error;
|
||||
|
||||
@@ -431,6 +431,10 @@ mod tests {
|
||||
fn header_by_number(&self, _num: u64) -> Result<Option<Header>> {
|
||||
Ok(self.parent.clone())
|
||||
}
|
||||
|
||||
fn header_td(&self, _hash: &BlockHash) -> Result<Option<U256>> {
|
||||
Ok(None)
|
||||
}
|
||||
}
|
||||
|
||||
fn mock_tx(nonce: u64) -> TransactionSignedEcRecovered {
|
||||
|
||||
@@ -364,6 +364,10 @@ impl HeaderProvider for MockEthProvider {
|
||||
let lock = self.headers.lock();
|
||||
Ok(lock.values().find(|h| h.number == num).cloned())
|
||||
}
|
||||
|
||||
fn header_td(&self, _hash: &BlockHash) -> reth_interfaces::Result<Option<U256>> {
|
||||
todo!()
|
||||
}
|
||||
}
|
||||
|
||||
impl BlockProvider for MockEthProvider {
|
||||
|
||||
@@ -11,7 +11,7 @@ Reth RPC types
|
||||
[dependencies]
|
||||
# reth
|
||||
reth-primitives = { path = "../../primitives" }
|
||||
reth-rlp = {path = "../../common/rlp"}
|
||||
reth-rlp = { path = "../../common/rlp" }
|
||||
|
||||
# misc
|
||||
serde = { version = "1.0", features = ["derive"] }
|
||||
|
||||
@@ -66,7 +66,7 @@ pub struct PayloadAttributes {
|
||||
/// Array of [`Withdrawal`] enabled with V2
|
||||
/// See <https://github.com/ethereum/execution-apis/blob/6709c2a795b707202e93c4f2867fa0bf2640a84f/src/engine/shanghai.md#executionpayloadv2>
|
||||
#[serde(default, skip_serializing_if = "Option::is_none")]
|
||||
pub withdrawal: Option<Withdrawal>,
|
||||
pub withdrawal: Option<Withdrawal>, // TODO: should be a vec
|
||||
}
|
||||
|
||||
/// This structure contains the result of processing a payload
|
||||
@@ -79,6 +79,16 @@ pub struct PayloadStatus {
|
||||
pub latest_valid_hash: Option<H256>,
|
||||
}
|
||||
|
||||
impl PayloadStatus {
|
||||
pub fn new(status: PayloadStatusEnum, latest_valid_hash: H256) -> Self {
|
||||
Self { status, latest_valid_hash: Some(latest_valid_hash) }
|
||||
}
|
||||
|
||||
pub fn from_status(status: PayloadStatusEnum) -> Self {
|
||||
Self { status, latest_valid_hash: None }
|
||||
}
|
||||
}
|
||||
|
||||
#[derive(Clone, Debug, PartialEq, Eq, Serialize, Deserialize)]
|
||||
#[serde(tag = "status", rename_all = "SCREAMING_SNAKE_CASE")]
|
||||
pub enum PayloadStatusEnum {
|
||||
@@ -96,7 +106,7 @@ pub enum PayloadStatusEnum {
|
||||
}
|
||||
|
||||
/// This structure contains configurable settings of the transition process.
|
||||
#[derive(Clone, Debug, PartialEq, Eq, Serialize, Deserialize)]
|
||||
#[derive(Default, Clone, Debug, PartialEq, Eq, Serialize, Deserialize)]
|
||||
#[serde(rename_all = "camelCase")]
|
||||
pub struct TransitionConfiguration {
|
||||
/// Maps on the TERMINAL_TOTAL_DIFFICULTY parameter of EIP-3675
|
||||
@@ -113,3 +123,23 @@ pub struct ForkchoiceUpdated {
|
||||
pub payload_status: PayloadStatus,
|
||||
pub payload_id: Option<H64>,
|
||||
}
|
||||
|
||||
impl ForkchoiceUpdated {
|
||||
pub fn new(payload_status: PayloadStatus) -> Self {
|
||||
Self { payload_status, payload_id: None }
|
||||
}
|
||||
|
||||
pub fn from_status(status: PayloadStatusEnum) -> Self {
|
||||
Self { payload_status: PayloadStatus::from_status(status), payload_id: None }
|
||||
}
|
||||
|
||||
pub fn with_latest_valid_hash(mut self, hash: H256) -> Self {
|
||||
self.payload_status.latest_valid_hash = Some(hash);
|
||||
self
|
||||
}
|
||||
|
||||
pub fn with_payload_id(mut self, id: H64) -> Self {
|
||||
self.payload_id = Some(id);
|
||||
self
|
||||
}
|
||||
}
|
||||
|
||||
@@ -14,15 +14,19 @@ reth-interfaces = { path = "../../interfaces" }
|
||||
reth-primitives = { path = "../../primitives" }
|
||||
reth-rpc-api = { path = "../rpc-api" }
|
||||
reth-rpc-types = { path = "../rpc-types" }
|
||||
reth-provider = { path = "../../storage/provider"}
|
||||
reth-provider = { path = "../../storage/provider" }
|
||||
reth-transaction-pool = { path = "../../transaction-pool" }
|
||||
reth-network = { path = "../network" }
|
||||
reth-consensus = { path = "../../consensus", features = ["serde"] }
|
||||
|
||||
# rpc
|
||||
jsonrpsee = { version = "0.16" }
|
||||
|
||||
# misc
|
||||
# async
|
||||
async-trait = "0.1"
|
||||
tokio = { version = "1", features = ["sync"] }
|
||||
|
||||
# misc
|
||||
serde = { version = "1.0", features = ["derive"] }
|
||||
serde_json = "1.0"
|
||||
thiserror = "1.0"
|
||||
|
||||
106
crates/net/rpc/src/engine/mod.rs
Normal file
106
crates/net/rpc/src/engine/mod.rs
Normal file
@@ -0,0 +1,106 @@
|
||||
use crate::result::rpc_err;
|
||||
use async_trait::async_trait;
|
||||
use jsonrpsee::core::{Error, RpcResult as Result};
|
||||
use reth_consensus::engine::{EngineApiError, EngineApiResult, EngineMessage};
|
||||
use reth_interfaces::consensus::ForkchoiceState;
|
||||
use reth_primitives::H64;
|
||||
use reth_rpc_api::EngineApiServer;
|
||||
use reth_rpc_types::engine::{
|
||||
ExecutionPayload, ForkchoiceUpdated, PayloadAttributes, PayloadStatus, TransitionConfiguration,
|
||||
};
|
||||
use tokio::sync::{
|
||||
mpsc::UnboundedSender,
|
||||
oneshot::{self, Receiver},
|
||||
};
|
||||
|
||||
/// The server implementation of Engine API
|
||||
pub struct EngineApi {
|
||||
/// Handle to the consensus engine
|
||||
engine_tx: UnboundedSender<EngineMessage>,
|
||||
}
|
||||
|
||||
impl std::fmt::Debug for EngineApi {
|
||||
fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
|
||||
f.debug_struct("EngineApi").finish_non_exhaustive()
|
||||
}
|
||||
}
|
||||
|
||||
impl EngineApi {
|
||||
async fn delegate_request<T>(
|
||||
&self,
|
||||
msg: EngineMessage,
|
||||
rx: Receiver<EngineApiResult<T>>,
|
||||
) -> Result<T> {
|
||||
let _ = self.engine_tx.send(msg);
|
||||
rx.await.map_err(|err| Error::Custom(err.to_string()))?.map_err(|err| {
|
||||
let code = match err {
|
||||
EngineApiError::PayloadUnknown => -38001,
|
||||
// Any other server error
|
||||
_ => jsonrpsee::types::error::INTERNAL_ERROR_CODE,
|
||||
};
|
||||
rpc_err(code, err.to_string(), None)
|
||||
})
|
||||
}
|
||||
}
|
||||
|
||||
#[async_trait]
|
||||
impl EngineApiServer for EngineApi {
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/8db51dcd2f4bdfbd9ad6e4a7560aac97010ad063/src/engine/specification.md#engine_newpayloadv1>
|
||||
/// Caution: This should not accept the `withdrawals` field
|
||||
async fn new_payload_v1(&self, payload: ExecutionPayload) -> Result<PayloadStatus> {
|
||||
let (tx, rx) = oneshot::channel();
|
||||
self.delegate_request(EngineMessage::NewPayload(payload, tx), rx).await
|
||||
}
|
||||
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/8db51dcd2f4bdfbd9ad6e4a7560aac97010ad063/src/engine/specification.md#engine_newpayloadv1>
|
||||
async fn new_payload_v2(&self, _payload: ExecutionPayload) -> Result<PayloadStatus> {
|
||||
todo!()
|
||||
}
|
||||
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/8db51dcd2f4bdfbd9ad6e4a7560aac97010ad063/src/engine/specification.md#engine_forkchoiceUpdatedV1>
|
||||
///
|
||||
/// Caution: This should not accept the `withdrawals` field
|
||||
async fn fork_choice_updated_v1(
|
||||
&self,
|
||||
fork_choice_state: ForkchoiceState,
|
||||
payload_attributes: Option<PayloadAttributes>,
|
||||
) -> Result<ForkchoiceUpdated> {
|
||||
let (tx, rx) = oneshot::channel();
|
||||
self.delegate_request(
|
||||
EngineMessage::ForkchoiceUpdated(fork_choice_state, payload_attributes, tx),
|
||||
rx,
|
||||
)
|
||||
.await
|
||||
}
|
||||
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/main/src/engine/specification.md#engine_forkchoiceupdatedv2>
|
||||
async fn fork_choice_updated_v2(
|
||||
&self,
|
||||
_fork_choice_state: ForkchoiceState,
|
||||
_payload_attributes: Option<PayloadAttributes>,
|
||||
) -> Result<ForkchoiceUpdated> {
|
||||
todo!()
|
||||
}
|
||||
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/8db51dcd2f4bdfbd9ad6e4a7560aac97010ad063/src/engine/specification.md#engine_getPayloadV1>
|
||||
///
|
||||
/// Caution: This should not return the `withdrawals` field
|
||||
async fn get_payload_v1(&self, payload_id: H64) -> Result<ExecutionPayload> {
|
||||
let (tx, rx) = oneshot::channel();
|
||||
self.delegate_request(EngineMessage::GetPayload(payload_id, tx), rx).await
|
||||
}
|
||||
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/main/src/engine/specification.md#engine_getpayloadv2>
|
||||
async fn get_payload_v2(&self, _payload_id: H64) -> Result<ExecutionPayload> {
|
||||
todo!()
|
||||
}
|
||||
|
||||
/// See also <https://github.com/ethereum/execution-apis/blob/8db51dcd2f4bdfbd9ad6e4a7560aac97010ad063/src/engine/specification.md#engine_exchangeTransitionConfigurationV1>
|
||||
async fn exchange_transition_configuration(
|
||||
&self,
|
||||
config: TransitionConfiguration,
|
||||
) -> Result<TransitionConfiguration> {
|
||||
let (tx, rx) = oneshot::channel();
|
||||
self.delegate_request(EngineMessage::ExchangeTransitionConfiguration(config, tx), rx).await
|
||||
}
|
||||
}
|
||||
@@ -1,8 +1,8 @@
|
||||
//! Provides everything related to `eth_` namespace
|
||||
|
||||
use reth_interfaces::Result;
|
||||
use reth_primitives::{U256, U64};
|
||||
use reth_provider::{BlockProvider, StateProviderFactory};
|
||||
use reth_primitives::U64;
|
||||
use reth_provider::{BlockProvider, ChainInfo, StateProviderFactory};
|
||||
use reth_rpc_types::Transaction;
|
||||
use reth_transaction_pool::TransactionPool;
|
||||
use std::sync::Arc;
|
||||
@@ -16,11 +16,11 @@ pub trait EthApiSpec: Send + Sync {
|
||||
/// Returns the current ethereum protocol version.
|
||||
fn protocol_version(&self) -> U64;
|
||||
|
||||
/// Returns the best block number
|
||||
fn block_number(&self) -> Result<U256>;
|
||||
|
||||
/// Returns the chain id
|
||||
fn chain_id(&self) -> U64;
|
||||
|
||||
/// Returns client chain info
|
||||
fn chain_info(&self) -> Result<ChainInfo>;
|
||||
}
|
||||
|
||||
/// `Eth` API implementation.
|
||||
@@ -66,15 +66,15 @@ where
|
||||
1u64.into()
|
||||
}
|
||||
|
||||
/// Returns the best block number
|
||||
fn block_number(&self) -> Result<U256> {
|
||||
Ok(self.client().chain_info()?.best_number.into())
|
||||
}
|
||||
|
||||
/// Returns the chain id
|
||||
fn chain_id(&self) -> U64 {
|
||||
todo!()
|
||||
}
|
||||
|
||||
/// Returns the current info for the chain
|
||||
fn chain_info(&self) -> Result<ChainInfo> {
|
||||
self.client().chain_info()
|
||||
}
|
||||
}
|
||||
|
||||
/// Container type `EthApi`
|
||||
|
||||
@@ -42,11 +42,14 @@ where
|
||||
}
|
||||
|
||||
fn block_number(&self) -> Result<U256> {
|
||||
EthApiSpec::block_number(self).with_message("Failed to read block number")
|
||||
Ok(EthApiSpec::chain_info(self)
|
||||
.with_message("failed to read chain info")?
|
||||
.best_number
|
||||
.into())
|
||||
}
|
||||
|
||||
async fn chain_id(&self) -> Result<Option<U64>> {
|
||||
todo!()
|
||||
Ok(Some(EthApiSpec::chain_id(self)))
|
||||
}
|
||||
|
||||
async fn block_by_hash(&self, _hash: H256, _full: bool) -> Result<Option<RichBlock>> {
|
||||
|
||||
@@ -11,9 +11,11 @@
|
||||
//!
|
||||
//! Provides the implementation of all RPC interfaces.
|
||||
|
||||
mod engine;
|
||||
mod eth;
|
||||
mod net;
|
||||
|
||||
pub use engine::EngineApi;
|
||||
pub use eth::{EthApi, EthApiSpec, EthPubSub};
|
||||
pub use net::NetApi;
|
||||
|
||||
|
||||
@@ -31,6 +31,9 @@ pub trait HeaderProvider: Send + Sync {
|
||||
BlockHashOrNumber::Number(num) => self.header_by_number(num),
|
||||
}
|
||||
}
|
||||
|
||||
/// Get total difficulty by block hash.
|
||||
fn header_td(&self, hash: &BlockHash) -> Result<Option<U256>>;
|
||||
}
|
||||
|
||||
/// Api trait for fetching `Block` related data.
|
||||
|
||||
@@ -1,10 +1,10 @@
|
||||
use crate::{BlockProvider, ChainInfo, HeaderProvider, ProviderImpl};
|
||||
use reth_db::{database::Database, tables, transaction::DbTx};
|
||||
use reth_interfaces::Result;
|
||||
use reth_primitives::{rpc::BlockId, Block, BlockNumber, Header, H256, U256};
|
||||
use reth_primitives::{rpc::BlockId, Block, BlockHash, BlockNumber, Header, H256, U256};
|
||||
|
||||
impl<DB: Database> HeaderProvider for ProviderImpl<DB> {
|
||||
fn header(&self, block_hash: &reth_primitives::BlockHash) -> Result<Option<Header>> {
|
||||
fn header(&self, block_hash: &BlockHash) -> Result<Option<Header>> {
|
||||
self.db.view(|tx| tx.get::<tables::Headers>((0, *block_hash).into()))?.map_err(Into::into)
|
||||
}
|
||||
|
||||
@@ -15,6 +15,15 @@ impl<DB: Database> HeaderProvider for ProviderImpl<DB> {
|
||||
Ok(None)
|
||||
}
|
||||
}
|
||||
|
||||
fn header_td(&self, hash: &BlockHash) -> Result<Option<U256>> {
|
||||
if let Some(num) = self.db.view(|tx| tx.get::<tables::HeaderNumbers>(*hash))?? {
|
||||
let td = self.db.view(|tx| tx.get::<tables::HeaderTD>((num, *hash).into()))??;
|
||||
Ok(td.map(|v| v.0))
|
||||
} else {
|
||||
Ok(None)
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
impl<DB: Database> BlockProvider for ProviderImpl<DB> {
|
||||
|
||||
@@ -39,4 +39,8 @@ impl HeaderProvider for TestApi {
|
||||
fn header_by_number(&self, _num: u64) -> Result<Option<Header>> {
|
||||
Ok(None)
|
||||
}
|
||||
|
||||
fn header_td(&self, _hash: &BlockHash) -> Result<Option<U256>> {
|
||||
Ok(None)
|
||||
}
|
||||
}
|
||||
|
||||
Reference in New Issue
Block a user