remove blob wrapper

This commit is contained in:
realbigsean 2022-11-19 15:18:42 -05:00
parent dfd0013eab
commit 45897ad4e1
No known key found for this signature in database
GPG Key ID: B372B64D866BF8CC
21 changed files with 199 additions and 159 deletions

View File

@ -102,12 +102,11 @@ use task_executor::{ShutdownReason, TaskExecutor};
use tree_hash::TreeHash;
use types::beacon_state::CloneConfig;
use types::*;
use types::signed_block_and_blobs::BlockMaybeBlobs;
pub type ForkChoiceError = fork_choice::Error<crate::ForkChoiceStoreError>;
/// Alias to appease clippy.
type HashBlockTuple<E> = (Hash256, BlockMaybeBlobs<E>);
type HashBlockTuple<E> = (Hash256, Arc<SignedBeaconBlock<E>>);
/// The time-out before failure during an operation to take a read/write RwLock on the block
/// processing cache.
@ -924,7 +923,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
pub async fn get_block(
&self,
block_root: &Hash256,
) -> Result<Option<SignedBeaconBlock<T::EthSpec>>, Error> {
) -> Result<Option<Arc<SignedBeaconBlock<T::EthSpec>>>, Error> {
// Load block from database, returning immediately if we have the full block w payload
// stored.
let blinded_block = match self.store.try_get_full_block(block_root)? {
@ -2198,7 +2197,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
/// This method is potentially long-running and should not run on the core executor.
pub fn filter_chain_segment(
self: &Arc<Self>,
chain_segment: Vec<BlockMaybeBlobs<T::EthSpec>>,
chain_segment: Vec<Arc<SignedBeaconBlock<T::EthSpec>>>,
) -> Result<Vec<HashBlockTuple<T::EthSpec>>, ChainSegmentResult<T::EthSpec>> {
// This function will never import any blocks.
let imported_blocks = 0;
@ -2304,7 +2303,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
/// `Self::process_block`.
pub async fn process_chain_segment(
self: &Arc<Self>,
chain_segment: Vec<BlockMaybeBlobs<T::EthSpec>>,
chain_segment: Vec<Arc<SignedBeaconBlock<T::EthSpec>>>,
count_unrealized: CountUnrealized,
) -> ChainSegmentResult<T::EthSpec> {
let mut imported_blocks = 0;
@ -2326,9 +2325,8 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
}
};
while let Some((_root, block_wrapper)) = filtered_chain_segment.first() {
let block: &SignedBeaconBlock<T::EthSpec> = block_wrapper.block();
while let Some((_root, block)) = filtered_chain_segment.first() {
let block: &SignedBeaconBlock<T::EthSpec> = block.block();
// Determine the epoch of the first block in the remaining segment.
let start_epoch = block.slot().epoch(T::EthSpec::slots_per_epoch());
@ -2406,17 +2404,17 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
/// Returns an `Err` if the given block was invalid, or an error was encountered during
pub async fn verify_block_for_gossip(
self: &Arc<Self>,
block_wrapper: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
) -> Result<GossipVerifiedBlock<T>, BlockError<T::EthSpec>> {
let chain = self.clone();
self.task_executor
.clone()
.spawn_blocking_handle(
move || {
let slot = block_wrapper.block().slot();
let graffiti_string = block_wrapper.block().message().body().graffiti().as_utf8_lossy();
let slot = block.block().slot();
let graffiti_string = block.block().message().body().graffiti().as_utf8_lossy();
match GossipVerifiedBlock::new(block_wrapper, &chain) {
match GossipVerifiedBlock::new(block, &chain) {
Ok(verified) => {
debug!(
chain.log,
@ -2618,7 +2616,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
#[allow(clippy::too_many_arguments)]
fn import_block(
&self,
block_wrapper: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
block_root: Hash256,
mut state: BeaconState<T::EthSpec>,
confirmed_state_roots: Vec<Hash256>,
@ -2627,7 +2625,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
parent_block: SignedBlindedBeaconBlock<T::EthSpec>,
parent_eth1_finalization_data: Eth1FinalizationData,
) -> Result<Hash256, BlockError<T::EthSpec>> {
let signed_block = block_wrapper.block();
let signed_block = block.block();
let current_slot = self.slot()?;
let current_epoch = current_slot.epoch(T::EthSpec::slots_per_epoch());
@ -2960,6 +2958,10 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
.collect();
ops.push(StoreOp::PutBlock(block_root, signed_block.clone()));
ops.push(StoreOp::PutState(block.state_root(), &state));
if let Some(blobs) = block.blobs() {
ops.push(StoreOp::PutBlobs(block_root, blobs));
};
let txn_lock = self.store.hot_db.begin_rw_transaction();
if let Err(e) = self.store.do_atomically(ops) {
@ -3366,7 +3368,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
//
// Wait for the execution layer to return an execution payload (if one is required).
let prepare_payload_handle = partial_beacon_block.prepare_payload_handle.take();
let execution_payload = if let Some(prepare_payload_handle) = prepare_payload_handle {
let block_contents = if let Some(prepare_payload_handle) = prepare_payload_handle {
prepare_payload_handle
.await
.map_err(BlockProductionError::TokioJoin)?
@ -3375,9 +3377,6 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
return Err(BlockProductionError::MissingExecutionPayload);
};
//FIXME(sean) waiting for the BN<>EE api for this to stabilize
let kzg_commitments = vec![];
// Part 3/3 (blocking)
//
// Perform the final steps of combining all the parts and computing the state root.
@ -3387,8 +3386,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
move || {
chain.complete_partial_beacon_block(
partial_beacon_block,
execution_payload,
kzg_commitments,
block_contents,
verification,
)
},
@ -3635,7 +3633,6 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
&self,
partial_beacon_block: PartialBeaconBlock<T::EthSpec, Payload>,
block_contents: BlockProposalContents<T::EthSpec, Payload>,
kzg_commitments: Vec<KzgCommitment>,
verification: ProduceBlockVerification,
) -> Result<BeaconBlockAndState<T::EthSpec, Payload>, BlockProductionError> {
let PartialBeaconBlock {
@ -3739,30 +3736,34 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
.map_err(|_| BlockProductionError::InvalidPayloadFork)?,
},
}),
BeaconState::Eip4844(_) => BeaconBlock::Eip4844(BeaconBlockEip4844 {
slot,
proposer_index,
parent_root,
state_root: Hash256::zero(),
body: BeaconBlockBodyEip4844 {
randao_reveal,
eth1_data,
graffiti,
proposer_slashings: proposer_slashings.into(),
attester_slashings: attester_slashings.into(),
attestations: attestations.into(),
deposits: deposits.into(),
voluntary_exits: voluntary_exits.into(),
sync_aggregate: sync_aggregate
.ok_or(BlockProductionError::MissingSyncAggregate)?,
execution_payload: block_contents
.to_payload()
.try_into()
.map_err(|_| BlockProductionError::InvalidPayloadFork)?,
//FIXME(sean) get blobs
blob_kzg_commitments: VariableList::from(kzg_commitments),
},
}),
BeaconState::Eip4844(_) => {
let kzg_commitments = block_contents
.kzg_commitments()
.ok_or(BlockProductionError::InvalidPayloadFork)?;
BeaconBlock::Eip4844(BeaconBlockEip4844 {
slot,
proposer_index,
parent_root,
state_root: Hash256::zero(),
body: BeaconBlockBodyEip4844 {
randao_reveal,
eth1_data,
graffiti,
proposer_slashings: proposer_slashings.into(),
attester_slashings: attester_slashings.into(),
attestations: attestations.into(),
deposits: deposits.into(),
voluntary_exits: voluntary_exits.into(),
sync_aggregate: sync_aggregate
.ok_or(BlockProductionError::MissingSyncAggregate)?,
execution_payload: block_contents
.to_payload()
.try_into()
.map_err(|_| BlockProductionError::InvalidPayloadFork)?,
blob_kzg_commitments: VariableList::from(kzg_commitments.to_vec()),
},
})
}
};
let block = SignedBeaconBlock::from_block(

View File

@ -1,6 +1,6 @@
use std::sync::Arc;
use derivative::Derivative;
use slot_clock::SlotClock;
use std::sync::Arc;
use crate::beacon_chain::{BeaconChain, BeaconChainTypes, MAXIMUM_GOSSIP_CLOCK_DISPARITY};
use crate::BeaconChainError;
@ -80,7 +80,10 @@ impl From<BeaconStateError> for BlobError {
}
}
pub fn validate_blob_for_gossip<T: BeaconChainTypes>(blob_sidecar: &BlobsSidecar<T::EthSpec>, chain: &Arc<BeaconChain<T>>) -> Result<(), BlobError>{
pub fn validate_blob_for_gossip<T: BeaconChainTypes>(
blob_sidecar: &BlobsSidecar<T::EthSpec>,
chain: &Arc<BeaconChain<T>>,
) -> Result<(), BlobError> {
let blob_slot = blob_sidecar.beacon_block_slot;
// Do not gossip or process blobs from future or past slots.
let latest_permissible_slot = chain
@ -121,4 +124,3 @@ pub fn validate_blob_for_gossip<T: BeaconChainTypes>(blob_sidecar: &BlobsSidecar
// TODO: `validate_blobs_sidecar`
Ok(())
}

View File

@ -42,6 +42,7 @@
//! END
//!
//! ```
use crate::blob_verification::validate_blob_for_gossip;
use crate::eth1_finalization_cache::Eth1FinalizationData;
use crate::execution_payload::{
is_optimistic_candidate_block, validate_execution_payload_for_gossip, validate_merge_block,
@ -83,14 +84,12 @@ use std::time::Duration;
use store::{Error as DBError, HotStateSummary, KeyValueStore, StoreOp};
use task_executor::JoinHandle;
use tree_hash::TreeHash;
use types::{BlobsSidecar, ExecPayload, SignedBeaconBlockAndBlobsSidecar};
use types::{
BeaconBlockRef, BeaconState, BeaconStateError, BlindedPayload, ChainSpec, CloneConfig, Epoch,
EthSpec, ExecutionBlockHash, Hash256, InconsistentFork, PublicKey, PublicKeyBytes,
RelativeEpoch, SignedBeaconBlock, SignedBeaconBlockHeader, Slot,
};
use types::signed_block_and_blobs::BlockMaybeBlobs;
use crate::blob_verification::validate_blob_for_gossip;
use types::{BlobsSidecar, ExecPayload, SignedBeaconBlockAndBlobsSidecar};
pub const POS_PANDA_BANNER: &str = r#"
,,, ,,, ,,, ,,,
@ -137,7 +136,7 @@ pub enum BlockError<T: EthSpec> {
///
/// It's unclear if this block is valid, but it cannot be processed without already knowing
/// its parent.
ParentUnknown(BlockMaybeBlobs<T>),
ParentUnknown(Arc<SignedBeaconBlock<T>>),
/// The block skips too many slots and is a DoS risk.
TooManySkippedSlots { parent_slot: Slot, block_slot: Slot },
/// The block slot is greater than the present slot.
@ -526,7 +525,7 @@ fn process_block_slash_info<T: BeaconChainTypes>(
/// The given `chain_segment` must contain only blocks from the same epoch, otherwise an error
/// will be returned.
pub fn signature_verify_chain_segment<T: BeaconChainTypes>(
mut chain_segment: Vec<(Hash256, BlockMaybeBlobs<T::EthSpec>)>,
mut chain_segment: Vec<(Hash256, Arc<SignedBeaconBlock<T::EthSpec>>)>,
chain: &BeaconChain<T>,
) -> Result<Vec<SignatureVerifiedBlock<T>>, BlockError<T::EthSpec>> {
if chain_segment.is_empty() {
@ -563,6 +562,8 @@ pub fn signature_verify_chain_segment<T: BeaconChainTypes>(
drop(pubkey_cache);
//FIXME(sean) batch verify kzg blobs
let mut signature_verified_blocks = chain_segment
.into_iter()
.map(|(block_root, block)| {
@ -591,7 +592,7 @@ pub fn signature_verify_chain_segment<T: BeaconChainTypes>(
#[derive(Derivative)]
#[derivative(Debug(bound = "T: BeaconChainTypes"))]
pub struct GossipVerifiedBlock<T: BeaconChainTypes> {
pub block: BlockMaybeBlobs<T::EthSpec>,
pub block: Arc<SignedBeaconBlock<T::EthSpec>>,
pub block_root: Hash256,
parent: Option<PreProcessingSnapshot<T::EthSpec>>,
consensus_context: ConsensusContext<T::EthSpec>,
@ -600,7 +601,7 @@ pub struct GossipVerifiedBlock<T: BeaconChainTypes> {
/// A wrapper around a `SignedBeaconBlock` that indicates that all signatures (except the deposit
/// signatures) have been verified.
pub struct SignatureVerifiedBlock<T: BeaconChainTypes> {
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
block_root: Hash256,
parent: Option<PreProcessingSnapshot<T::EthSpec>>,
consensus_context: ConsensusContext<T::EthSpec>,
@ -617,12 +618,13 @@ type PayloadVerificationHandle<E> =
/// - Signatures
/// - State root check
/// - Per block processing
/// - Blobs sidecar has been validated if present
///
/// Note: a `ExecutionPendingBlock` is not _forever_ valid to be imported, it may later become invalid
/// due to finality or some other event. A `ExecutionPendingBlock` should be imported into the
/// `BeaconChain` immediately after it is instantiated.
pub struct ExecutionPendingBlock<T: BeaconChainTypes> {
pub block: BlockMaybeBlobs<T::EthSpec>,
pub block: Arc<SignedBeaconBlock<T::EthSpec>>,
pub block_root: Hash256,
pub state: BeaconState<T::EthSpec>,
pub parent_block: SignedBeaconBlock<T::EthSpec, BlindedPayload<T::EthSpec>>,
@ -667,7 +669,7 @@ impl<T: BeaconChainTypes> GossipVerifiedBlock<T> {
///
/// Returns an error if the block is invalid, or if the block was unable to be verified.
pub fn new(
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
chain: &BeaconChain<T>,
) -> Result<Self, BlockError<T::EthSpec>> {
// If the block is valid for gossip we don't supply it to the slasher here because
@ -682,10 +684,10 @@ impl<T: BeaconChainTypes> GossipVerifiedBlock<T> {
/// As for new, but doesn't pass the block to the slasher.
fn new_without_slasher_checks(
block_wrapper: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
chain: &BeaconChain<T>,
) -> Result<Self, BlockError<T::EthSpec>> {
let block = block_wrapper.block();
let block = block.block();
// Ensure the block is the correct structure for the fork at `block.slot()`.
block
.fork_name(&chain.spec)
@ -879,9 +881,10 @@ impl<T: BeaconChainTypes> GossipVerifiedBlock<T> {
// Validate the block's execution_payload (if any).
validate_execution_payload_for_gossip(&parent_block, block.message(), chain)?;
if let Some(blobs_sidecar) = block_wrapper.blobs() {
validate_blob_for_gossip(blobs_sidecar, chain)?;
}
//FIXME(sean)
// if let Some(blobs_sidecar) = block.blobs() {
// validate_blob_for_gossip(blobs_sidecar, chain)?;
// }
// Having checked the proposer index and the block root we can cache them.
let consensus_context = ConsensusContext::new(block.slot())
@ -889,7 +892,7 @@ impl<T: BeaconChainTypes> GossipVerifiedBlock<T> {
.set_proposer_index(block.message().proposer_index());
Ok(Self {
block: block_wrapper,
block: block,
block_root,
parent,
consensus_context,
@ -924,7 +927,7 @@ impl<T: BeaconChainTypes> SignatureVerifiedBlock<T> {
///
/// Returns an error if the block is invalid, or if the block was unable to be verified.
pub fn new(
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
block_root: Hash256,
chain: &BeaconChain<T>,
) -> Result<Self, BlockError<T::EthSpec>> {
@ -970,7 +973,7 @@ impl<T: BeaconChainTypes> SignatureVerifiedBlock<T> {
/// As for `new` above but producing `BlockSlashInfo`.
pub fn check_slashable(
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
block_root: Hash256,
chain: &BeaconChain<T>,
) -> Result<Self, BlockSlashInfo<BlockError<T::EthSpec>>> {
@ -1064,7 +1067,7 @@ impl<T: BeaconChainTypes> IntoExecutionPendingBlock<T> for SignatureVerifiedBloc
}
}
impl<T: BeaconChainTypes> IntoExecutionPendingBlock<T> for BlockMaybeBlobs<T::EthSpec> {
impl<T: BeaconChainTypes> IntoExecutionPendingBlock<T> for Arc<SignedBeaconBlock<T::EthSpec>> {
/// Verifies the `SignedBeaconBlock` by first transforming it into a `SignatureVerifiedBlock`
/// and then using that implementation of `IntoExecutionPendingBlock` to complete verification.
fn into_execution_pending_block_slashable(
@ -1097,13 +1100,13 @@ impl<T: BeaconChainTypes> ExecutionPendingBlock<T> {
///
/// Returns an error if the block is invalid, or if the block was unable to be verified.
pub fn from_signature_verified_components(
block_wrapper: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
block_root: Hash256,
parent: PreProcessingSnapshot<T::EthSpec>,
mut consensus_context: ConsensusContext<T::EthSpec>,
chain: &Arc<BeaconChain<T>>,
) -> Result<Self, BlockError<T::EthSpec>> {
let block = block_wrapper.block();
let block = block.block();
if let Some(parent) = chain
.canonical_head
@ -1128,7 +1131,7 @@ impl<T: BeaconChainTypes> ExecutionPendingBlock<T> {
// because it will revert finalization. Note that the finalized block is stored in fork
// choice, so we will not reject any child of the finalized block (this is relevant during
// genesis).
return Err(BlockError::ParentUnknown(block_wrapper));
return Err(BlockError::ParentUnknown(block));
}
// Reject any block that exceeds our limit on skipped slots.
@ -1545,7 +1548,7 @@ pub fn check_block_is_finalized_descendant<T: BeaconChainTypes>(
})
} else {
//FIXME(sean) does this matter if it only returns a block?
Err(BlockError::ParentUnknown(BlockMaybeBlobs::Block(block.clone())))
Err(BlockError::ParentUnknown(block.clone()))
}
}
}
@ -1637,16 +1640,15 @@ fn verify_parent_block_is_known<T: BeaconChainTypes>(
#[allow(clippy::type_complexity)]
fn load_parent<T: BeaconChainTypes>(
block_root: Hash256,
block_wrapper: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
chain: &BeaconChain<T>,
) -> Result<
(
PreProcessingSnapshot<T::EthSpec>,
BlockMaybeBlobs<T::EthSpec>,
Arc<SignedBeaconBlock<T::EthSpec>>,
),
BlockError<T::EthSpec>,
> {
let block = block_wrapper.block();
let spec = &chain.spec;
// Reject any block if its parent is not known to fork choice.
@ -1664,7 +1666,7 @@ fn load_parent<T: BeaconChainTypes>(
.fork_choice_read_lock()
.contains_block(&block.parent_root())
{
return Err(BlockError::ParentUnknown(block_wrapper));
return Err(BlockError::ParentUnknown(block));
}
let block_delay = chain
@ -1703,7 +1705,7 @@ fn load_parent<T: BeaconChainTypes>(
"block_delay" => ?block_delay,
);
}
Ok((snapshot, block_wrapper))
Ok((snapshot, block))
} else {
// Load the blocks parent block from the database, returning invalid if that block is not
// found.
@ -1750,7 +1752,7 @@ fn load_parent<T: BeaconChainTypes>(
pre_state: parent_state,
beacon_state_root: Some(parent_state_root),
},
block_wrapper,
beacon_block,
))
};

View File

@ -15,7 +15,11 @@ use std::io::{Read, Write};
use std::marker::PhantomData;
use std::sync::Arc;
use tokio_util::codec::{Decoder, Encoder};
use types::{BlobsSidecar, EthSpec, ForkContext, ForkName, SignedBeaconBlock, SignedBeaconBlockAltair, SignedBeaconBlockAndBlobsSidecar, SignedBeaconBlockBase, SignedBeaconBlockCapella, SignedBeaconBlockEip4844, SignedBeaconBlockMerge};
use types::{
BlobsSidecar, EthSpec, ForkContext, ForkName, SignedBeaconBlock, SignedBeaconBlockAltair,
SignedBeaconBlockAndBlobsSidecar, SignedBeaconBlockBase, SignedBeaconBlockCapella,
SignedBeaconBlockEip4844, SignedBeaconBlockMerge,
};
use unsigned_varint::codec::Uvi;
const CONTEXT_BYTES_LEN: usize = 4;
@ -311,8 +315,11 @@ impl<TSpec: EthSpec> Decoder for SSZSnappyOutboundCodec<TSpec> {
let _read_bytes = src.split_to(n as usize);
match self.protocol.version {
Version::V1 => handle_v1_response(self.protocol.message_name, &decoded_buffer,
&mut self.fork_name, ),
Version::V1 => handle_v1_response(
self.protocol.message_name,
&decoded_buffer,
&mut self.fork_name,
),
Version::V2 => handle_v2_response(
self.protocol.message_name,
&decoded_buffer,
@ -482,11 +489,9 @@ fn handle_v1_request<T: EthSpec>(
Protocol::BlobsByRange => Ok(Some(InboundRequest::BlobsByRange(
BlobsByRangeRequest::from_ssz_bytes(decoded_buffer)?,
))),
Protocol::BlobsByRoot => Ok(Some(InboundRequest::BlobsByRoot(
BlobsByRootRequest{
block_roots: VariableList::from_ssz_bytes(decoded_buffer)?,
},
))),
Protocol::BlobsByRoot => Ok(Some(InboundRequest::BlobsByRoot(BlobsByRootRequest {
block_roots: VariableList::from_ssz_bytes(decoded_buffer)?,
}))),
Protocol::Ping => Ok(Some(InboundRequest::Ping(Ping {
data: u64::from_ssz_bytes(decoded_buffer)?,
}))),
@ -576,7 +581,7 @@ fn handle_v1_response<T: EthSpec>(
"Invalid forkname for blobsbyrange".to_string(),
)),
}
},
}
Protocol::BlobsByRoot => {
let fork_name = fork_name.take().ok_or_else(|| {
RPCError::ErrorResponse(
@ -593,7 +598,7 @@ fn handle_v1_response<T: EthSpec>(
"Invalid forkname for blobsbyroot".to_string(),
)),
}
},
}
Protocol::Ping => Ok(Some(RPCResponse::Pong(Ping {
data: u64::from_ssz_bytes(decoded_buffer)?,
}))),
@ -678,8 +683,12 @@ fn handle_v2_response<T: EthSpec>(
)?),
)))),
},
Protocol::BlobsByRange => Err(RPCError::InvalidData("blobs by range via v2".to_string())),
Protocol::BlobsByRoot => Err(RPCError::InvalidData("blobs by range via v2".to_string())),
Protocol::BlobsByRange => {
Err(RPCError::InvalidData("blobs by range via v2".to_string()))
}
Protocol::BlobsByRoot => {
Err(RPCError::InvalidData("blobs by range via v2".to_string()))
}
_ => Err(RPCError::ErrorResponse(
RPCResponseErrorCode::InvalidRequest,
"Invalid v2 request".to_string(),

View File

@ -13,8 +13,8 @@ use std::sync::Arc;
use strum::IntoStaticStr;
use superstruct::superstruct;
use types::blobs_sidecar::BlobsSidecar;
use types::{Epoch, EthSpec, Hash256, SignedBeaconBlock, Slot};
use types::SignedBeaconBlockAndBlobsSidecar;
use types::{Epoch, EthSpec, Hash256, SignedBeaconBlock, Slot};
/// Maximum number of blocks in a single request.
pub type MaxRequestBlocks = U1024;
@ -427,10 +427,18 @@ impl<T: EthSpec> std::fmt::Display for RPCResponse<T> {
write!(f, "BlocksByRoot: Block slot: {}", block.slot())
}
RPCResponse::BlobsByRange(blob) => {
write!(f, "BlobsByRange: Blob slot: {}", blob.blobs_sidecar.beacon_block_slot)
write!(
f,
"BlobsByRange: Blob slot: {}",
blob.blobs_sidecar.beacon_block_slot
)
}
RPCResponse::BlobsByRoot(blob) => {
write!(f, "BlobsByRoot: Blob slot: {}", blob.blobs_sidecar.beacon_block_slot)
write!(
f,
"BlobsByRoot: Blob slot: {}",
blob.blobs_sidecar.beacon_block_slot
)
}
RPCResponse::Pong(ping) => write!(f, "Pong: {}", ping.data),
RPCResponse::MetaData(metadata) => write!(f, "Metadata: {}", metadata.seq_number()),

View File

@ -82,9 +82,11 @@ impl<TSpec: EthSpec> OutboundRequest<TSpec> {
Version::V1,
Encoding::SSZSnappy,
)],
OutboundRequest::BlobsByRoot(_) => vec![
ProtocolId::new(Protocol::BlobsByRoot, Version::V1, Encoding::SSZSnappy),
],
OutboundRequest::BlobsByRoot(_) => vec![ProtocolId::new(
Protocol::BlobsByRoot,
Version::V1,
Encoding::SSZSnappy,
)],
OutboundRequest::Ping(_) => vec![ProtocolId::new(
Protocol::Ping,
Version::V1,

View File

@ -1264,11 +1264,8 @@ impl<AppReqId: ReqId, TSpec: EthSpec> Network<AppReqId, TSpec> {
Some(event)
}
InboundRequest::BlobsByRoot(req) => {
let event = self.build_request(
peer_request_id,
peer_id,
Request::BlobsByRoot(req),
);
let event =
self.build_request(peer_request_id, peer_id, Request::BlobsByRoot(req));
Some(event)
}
}

View File

@ -11,7 +11,13 @@ use std::boxed::Box;
use std::io::{Error, ErrorKind};
use std::sync::Arc;
use tree_hash_derive::TreeHash;
use types::{Attestation, AttesterSlashing, BlobsSidecar, EthSpec, ForkContext, ForkName, ProposerSlashing, SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAltair, SignedBeaconBlockAndBlobsSidecar, SignedBeaconBlockBase, SignedBeaconBlockCapella, SignedBeaconBlockEip4844, SignedBeaconBlockMerge, SignedContributionAndProof, SignedVoluntaryExit, SubnetId, SyncCommitteeMessage, SyncSubnetId};
use types::{
Attestation, AttesterSlashing, BlobsSidecar, EthSpec, ForkContext, ForkName, ProposerSlashing,
SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAltair,
SignedBeaconBlockAndBlobsSidecar, SignedBeaconBlockBase, SignedBeaconBlockCapella,
SignedBeaconBlockEip4844, SignedBeaconBlockMerge, SignedContributionAndProof,
SignedVoluntaryExit, SubnetId, SyncCommitteeMessage, SyncSubnetId,
};
#[derive(Debug, Clone, PartialEq)]
pub enum PubsubMessage<T: EthSpec> {
@ -281,7 +287,7 @@ impl<T: EthSpec> std::fmt::Display for PubsubMessage<T> {
PubsubMessage::BeaconBlockAndBlobsSidecars(block_and_blob) => write!(
f,
"Beacon block and Blobs Sidecar: slot: {}, blobs: {}",
block_and_blob.beacon_block.message().slot(),
block_and_blob.beacon_block.message.slot,
block_and_blob.blobs_sidecar.blobs.len(),
),
PubsubMessage::AggregateAndProofAttestation(att) => write!(

View File

@ -61,7 +61,11 @@ use std::time::Duration;
use std::{cmp, collections::HashSet};
use task_executor::TaskExecutor;
use tokio::sync::mpsc;
use types::{Attestation, AttesterSlashing, Hash256, ProposerSlashing, SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar, SignedContributionAndProof, SignedVoluntaryExit, SubnetId, SyncCommitteeMessage, SyncSubnetId};
use types::{
Attestation, AttesterSlashing, Hash256, ProposerSlashing, SignedAggregateAndProof,
SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar, SignedContributionAndProof,
SignedVoluntaryExit, SubnetId, SyncCommitteeMessage, SyncSubnetId,
};
use work_reprocessing_queue::{
spawn_reprocess_scheduler, QueuedAggregate, QueuedRpcBlock, QueuedUnaggregate, ReadyWork,
};

View File

@ -10,9 +10,7 @@ use beacon_chain::{
BeaconChainError, BeaconChainTypes, BlockError, CountUnrealized, ForkChoiceError,
GossipVerifiedBlock,
};
use lighthouse_network::{
Client, MessageAcceptance, MessageId, PeerAction, PeerId, ReportSource,
};
use lighthouse_network::{Client, MessageAcceptance, MessageId, PeerAction, PeerId, ReportSource};
use slog::{crit, debug, error, info, trace, warn};
use slot_clock::SlotClock;
use ssz::Encode;
@ -20,8 +18,12 @@ use std::sync::Arc;
use std::time::{Duration, SystemTime, UNIX_EPOCH};
use store::hot_cold_store::HotColdDBError;
use tokio::sync::mpsc;
use types::{Attestation, AttesterSlashing, BlobsSidecar, EthSpec, Hash256, IndexedAttestation, ProposerSlashing, SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar, SignedContributionAndProof, SignedVoluntaryExit, Slot, SubnetId, SyncCommitteeMessage, SyncSubnetId};
use types::signed_block_and_blobs::BlockMaybeBlobs;
use types::{
Attestation, AttesterSlashing, BlobsSidecar, EthSpec, Hash256, IndexedAttestation,
ProposerSlashing, SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar,
SignedContributionAndProof, SignedVoluntaryExit, Slot, SubnetId, SyncCommitteeMessage,
SyncSubnetId,
};
use super::{
super::work_reprocessing_queue::{
@ -655,7 +657,7 @@ impl<T: BeaconChainTypes> Worker<T> {
message_id: MessageId,
peer_id: PeerId,
peer_client: Client,
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
reprocess_tx: mpsc::Sender<ReprocessQueueMessage<T>>,
duplicate_cache: DuplicateCache,
seen_duration: Duration,
@ -702,7 +704,7 @@ impl<T: BeaconChainTypes> Worker<T> {
message_id: MessageId,
peer_id: PeerId,
peer_client: Client,
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
reprocess_tx: mpsc::Sender<ReprocessQueueMessage<T>>,
seen_duration: Duration,
) -> Option<GossipVerifiedBlock<T>> {

View File

@ -7,7 +7,7 @@ use crate::sync::manager::RequestId as SyncId;
use crate::sync::SyncMessage;
use beacon_chain::{BeaconChain, BeaconChainTypes};
use lighthouse_network::rpc::methods::{BlobsByRangeRequest, BlobsByRootRequest};
use lighthouse_network::{rpc::*};
use lighthouse_network::rpc::*;
use lighthouse_network::{
Client, MessageId, NetworkGlobals, PeerId, PeerRequestId, Request, Response,
};
@ -17,7 +17,11 @@ use std::sync::Arc;
use std::time::{Duration, SystemTime, UNIX_EPOCH};
use store::SyncCommitteeMessage;
use tokio::sync::mpsc;
use types::{Attestation, AttesterSlashing, BlobsSidecar, EthSpec, ProposerSlashing, SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar, SignedContributionAndProof, SignedVoluntaryExit, SubnetId, SyncSubnetId};
use types::{
Attestation, AttesterSlashing, BlobsSidecar, EthSpec, ProposerSlashing,
SignedAggregateAndProof, SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar,
SignedContributionAndProof, SignedVoluntaryExit, SubnetId, SyncSubnetId,
};
/// Processes validated messages from the network. It relays necessary data to the syncing thread
/// and processes blocks from the pubsub network.

View File

@ -9,7 +9,6 @@ use slog::{debug, error, trace, warn, Logger};
use smallvec::SmallVec;
use std::sync::Arc;
use store::{Hash256, SignedBeaconBlock};
use types::signed_block_and_blobs::BlockMaybeBlobs;
use crate::beacon_processor::{ChainSegmentProcessId, WorkEvent};
use crate::metrics;
@ -31,7 +30,7 @@ mod single_block_lookup;
#[cfg(test)]
mod tests;
pub type RootBlockTuple<T> = (Hash256, BlockMaybeBlobs<T>);
pub type RootBlockTuple<T> = (Hash256, Arc<SignedBeaconBlock<T>>);
const FAILED_CHAINS_CACHE_EXPIRY_SECONDS: u64 = 60;
const SINGLE_BLOCK_LOOKUP_MAX_ATTEMPTS: u8 = 3;

View File

@ -4,7 +4,6 @@ use lighthouse_network::PeerId;
use std::sync::Arc;
use store::{Hash256, SignedBeaconBlock};
use strum::IntoStaticStr;
use types::signed_block_and_blobs::BlockMaybeBlobs;
use crate::sync::{
manager::{Id, SLOT_IMPORT_TOLERANCE},
@ -25,7 +24,7 @@ pub(crate) struct ParentLookup<T: BeaconChainTypes> {
/// The root of the block triggering this parent request.
chain_hash: Hash256,
/// The blocks that have currently been downloaded.
downloaded_blocks: Vec<BlockMaybeBlobs<T::EthSpec>>,
downloaded_blocks: Vec<Arc<SignedBeaconBlock<T::EthSpec>>>,
/// Request of the last parent.
current_parent_request: SingleBlockRequest<PARENT_FAIL_TOLERANCE>,
/// Id of the last parent request.
@ -62,7 +61,7 @@ impl<T: BeaconChainTypes> ParentLookup<T> {
pub fn new(
block_root: Hash256,
block: BlockMaybeBlobs<T::EthSpec>,
block: Arc<SignedBeaconBlock<T::EthSpec>>,
peer_id: PeerId,
) -> Self {
let current_parent_request = SingleBlockRequest::new(block.parent_root(), peer_id);
@ -99,7 +98,7 @@ impl<T: BeaconChainTypes> ParentLookup<T> {
self.current_parent_request.check_peer_disconnected(peer_id)
}
pub fn add_block(&mut self, block: BlockMaybeBlobs<T::EthSpec>) {
pub fn add_block(&mut self, block: Arc<SignedBeaconBlock<T::EthSpec>>) {
let next_parent = block.parent_root();
self.downloaded_blocks.push(block);
self.current_parent_request.hash = next_parent;
@ -126,7 +125,7 @@ impl<T: BeaconChainTypes> ParentLookup<T> {
self.current_parent_request_id = None;
}
pub fn chain_blocks(&mut self) -> Vec<BlockMaybeBlobs<T::EthSpec>> {
pub fn chain_blocks(&mut self) -> Vec<Arc<SignedBeaconBlock<T::EthSpec>>> {
std::mem::take(&mut self.downloaded_blocks)
}
@ -134,7 +133,7 @@ impl<T: BeaconChainTypes> ParentLookup<T> {
/// the processing result of the block.
pub fn verify_block(
&mut self,
block: Option<BlockMaybeBlobs<T::EthSpec>>,
block: Option<Arc<SignedBeaconBlock<T::EthSpec>>>,
failed_chains: &mut lru_cache::LRUTimeCache<Hash256>,
) -> Result<Option<RootBlockTuple<T::EthSpec>>, VerifyError> {
let root_and_block = self.current_parent_request.verify_block(block)?;

View File

@ -8,7 +8,6 @@ use rand::seq::IteratorRandom;
use ssz_types::VariableList;
use store::{EthSpec, Hash256, SignedBeaconBlock};
use strum::IntoStaticStr;
use types::signed_block_and_blobs::BlockMaybeBlobs;
/// Object representing a single block lookup request.
#[derive(PartialEq, Eq)]
@ -106,7 +105,7 @@ impl<const MAX_ATTEMPTS: u8> SingleBlockRequest<MAX_ATTEMPTS> {
/// Returns the block for processing if the response is what we expected.
pub fn verify_block<T: EthSpec>(
&mut self,
block: Option<BlockMaybeBlobs<T>>,
block: Option<Arc<SignedBeaconBlock<T>>>,
) -> Result<Option<RootBlockTuple<T>>, VerifyError> {
match self.state {
State::AwaitingDownload => {

View File

@ -53,7 +53,6 @@ use std::ops::Sub;
use std::sync::Arc;
use std::time::Duration;
use tokio::sync::mpsc;
use types::signed_block_and_blobs::BlockMaybeBlobs;
use types::{
BlobsSidecar, EthSpec, Hash256, SignedBeaconBlock, SignedBeaconBlockAndBlobsSidecar, Slot,
};
@ -104,12 +103,12 @@ pub enum SyncMessage<T: EthSpec> {
RpcBlock {
request_id: RequestId,
peer_id: PeerId,
beacon_block: Option<BlockMaybeBlobs<T>>,
beacon_block: Option<Arc<SignedBeaconBlock<T>>>,
seen_timestamp: Duration,
},
/// A block with an unknown parent has been received.
UnknownBlock(PeerId, BlockMaybeBlobs<T>, Hash256),
UnknownBlock(PeerId, Arc<SignedBeaconBlock<T>>, Hash256),
/// A peer has sent an object that references a block that is unknown. This triggers the
/// manager to attempt to find the block matching the unknown hash.

View File

@ -1,8 +1,8 @@
use std::marker::PhantomData;
use tree_hash::TreeHash;
use types::{
AbstractExecPayload, BeaconState, BeaconStateError, ChainSpec, EthSpec, ExecPayload, Hash256,
SignedBeaconBlock, Slot,
AbstractExecPayload, BeaconState, BeaconStateError, BlobsSidecar, ChainSpec, EthSpec,
ExecPayload, Hash256, SignedBeaconBlock, Slot,
};
#[derive(Debug)]
@ -13,6 +13,12 @@ pub struct ConsensusContext<T: EthSpec> {
proposer_index: Option<u64>,
/// Block root of the block at `slot`.
current_block_root: Option<Hash256>,
/// Should only be populated if the sidecar has not been validated.
blobs_sidecar: Option<Box<BlobsSidecar<T>>>,
/// Whether `validate_blobs_sidecar` has successfully passed.
blobs_sidecar_validated: bool,
/// Whether `verify_kzg_commitments_against_transactions` has successfully passed.
blobs_verified_vs_txs: bool,
_phantom: PhantomData<T>,
}
@ -34,6 +40,9 @@ impl<T: EthSpec> ConsensusContext<T> {
slot,
proposer_index: None,
current_block_root: None,
blobs_sidecar: None,
blobs_sidecar_validated: false,
blobs_verified_vs_txs: false,
_phantom: PhantomData,
}
}
@ -89,4 +98,22 @@ impl<T: EthSpec> ConsensusContext<T> {
})
}
}
pub fn set_blobs_sidecar_validated(mut self, blobs_sidecar_validated: bool) -> Self {
self.blobs_sidecar_validated = blobs_sidecar_validated;
self
}
pub fn set_blobs_verified_vs_txs(mut self, blobs_verified_vs_txs: bool) -> Self {
self.blobs_verified_vs_txs = blobs_verified_vs_txs;
self
}
pub fn blobs_sidecar_validated(&self) -> bool {
self.blobs_sidecar_validated
}
pub fn blobs_verified_vs_txs(&self) -> bool {
self.blobs_verified_vs_txs
}
}

View File

@ -180,6 +180,9 @@ pub fn per_block_processing<T: EthSpec, Payload: AbstractExecPayload<T>>(
process_blob_kzg_commitments(block.body())?;
//FIXME(sean) add `validate_blobs_sidecar` (is_data_available) and only run it if the consensus
// context tells us it wasnt already run
Ok(())
}

View File

@ -18,6 +18,7 @@ pub fn process_blob_kzg_commitments<T: EthSpec, Payload: AbstractExecPayload<T>>
block_body.blob_kzg_commitments(),
) {
if let Some(transactions) = payload.transactions() {
//FIXME(sean) only run if this wasn't run in gossip (use consensus context)
if !verify_kzg_commitments_against_transactions::<T>(transactions, kzg_commitments)? {
return Err(BlockProcessingError::BlobVersionHashMismatch);
}

View File

@ -11,7 +11,7 @@ pub fn upgrade_to_eip4844<E: EthSpec>(
// FIXME(sean) This is a hack to let us participate in testnets where capella doesn't exist.
// if we are disabling withdrawals, assume we should fork off of bellatrix.
let previous_fork_version = if cfg!(feature ="withdrawals") {
let previous_fork_version = if cfg!(feature = "withdrawals") {
pre.fork.current_version
} else {
spec.bellatrix_fork_version

View File

@ -150,7 +150,6 @@ pub use crate::historical_batch::HistoricalBatch;
pub use crate::indexed_attestation::IndexedAttestation;
pub use crate::kzg_commitment::KzgCommitment;
pub use crate::kzg_proof::KzgProof;
pub use crate::signed_block_and_blobs::SignedBeaconBlockAndBlobsSidecar;
pub use crate::participation_flags::ParticipationFlags;
pub use crate::participation_list::ParticipationList;
pub use crate::payload::{
@ -172,6 +171,7 @@ pub use crate::signed_beacon_block::{
SignedBlindedBeaconBlock,
};
pub use crate::signed_beacon_block_header::SignedBeaconBlockHeader;
pub use crate::signed_block_and_blobs::SignedBeaconBlockAndBlobsSidecar;
pub use crate::signed_contribution_and_proof::SignedContributionAndProof;
pub use crate::signed_voluntary_exit::SignedVoluntaryExit;
pub use crate::signing_data::{SignedRoot, SigningData};

View File

@ -1,37 +1,13 @@
use std::sync::Arc;
use serde_derive::{Deserialize, Serialize};
use ssz::{Decode, DecodeError};
use ssz_derive::{Decode, Encode};
use tree_hash_derive::TreeHash;
use crate::{BlobsSidecar, EthSpec, SignedBeaconBlock, SignedBeaconBlockEip4844};
use serde_derive::{Deserialize, Serialize};
use ssz::{Decode, DecodeError, Encode};
use ssz_derive::{Decode, Encode};
use std::sync::Arc;
use tree_hash_derive::TreeHash;
#[derive(Debug, Clone, Serialize, Deserialize, Encode, TreeHash, PartialEq)]
#[derive(Debug, Clone, Serialize, Deserialize, Encode, Decode, TreeHash, PartialEq)]
#[serde(bound = "T: EthSpec")]
pub struct SignedBeaconBlockAndBlobsSidecar<T: EthSpec> {
pub beacon_block: SignedBeaconBlock<T>,
pub beacon_block: SignedBeaconBlockEip4844<T>,
pub blobs_sidecar: BlobsSidecar<T>,
}
impl <T: EthSpec>Decode for SignedBeaconBlockAndBlobsSidecar<T> {
fn is_ssz_fixed_len() -> bool {
todo!()
}
fn from_ssz_bytes(bytes: &[u8]) -> Result<Self, DecodeError> {
todo!()
}
}
pub enum BlockMaybeBlobs<T: EthSpec> {
Block(Arc<SignedBeaconBlock<T>>),
BlockAndBlobs(Arc<SignedBeaconBlockAndBlobsSidecar<T>>),
}
impl <T: EthSpec> BlockMaybeBlobs<T> {
pub fn blobs(&self) -> Option<&BlobsSidecar<T>>{
match self {
Self::Block(_) => None,
Self::BlockAndBlobs(block_and_blobs) => Some(&block_and_blobs.blobs_sidecar)
}
}
}