Implement Overflow LRU Cache for Pending Blobs (#4203)

* All Necessary Objects Implement Encode/Decode

* Major Components for LRUOverflowCache Implemented

* Finish Database Code

* Add Maintenance Methods

* Added Maintenance Service

* Persist Blobs on Shutdown / Reload on Startup

* Address Clippy Complaints

* Add (emum_behaviour = "tag") to ssz_derive

* Convert Encode/Decode Implementations to "tag"

* Started Adding Tests

* Added a ton of tests

* 1 character fix

* Feature Guard Minimal Spec Tests

* Update beacon_node/beacon_chain/src/data_availability_checker.rs

Co-authored-by: realbigsean <seananderson33@GMAIL.com>

* Address Sean's Comments

* Add iter_raw_keys method

* Remove TODOs

---------

Co-authored-by: realbigsean <seananderson33@GMAIL.com>
This commit is contained in:
ethDreamer 2023-05-12 09:08:24 -05:00 committed by GitHub
parent a22e4bf636
commit 46db30416d
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
23 changed files with 2364 additions and 253 deletions

View File

@ -465,7 +465,7 @@ pub struct BeaconChain<T: BeaconChainTypes> {
/// Provides monitoring of a set of explicitly defined validators.
pub validator_monitor: RwLock<ValidatorMonitor<T::EthSpec>>,
pub proposal_blob_cache: BlobCache<T::EthSpec>,
pub data_availability_checker: DataAvailabilityChecker<T::EthSpec, T::SlotClock>,
pub data_availability_checker: DataAvailabilityChecker<T>,
pub kzg: Option<Arc<Kzg>>,
}
@ -609,6 +609,13 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
Ok(())
}
pub fn persist_data_availabilty_checker(&self) -> Result<(), Error> {
let _timer = metrics::start_timer(&metrics::PERSIST_DATA_AVAILABILITY_CHECKER);
self.data_availability_checker.persist_all()?;
Ok(())
}
/// Returns the slot _right now_ according to `self.slot_clock`. Returns `Err` if the slot is
/// unavailable.
///
@ -6268,6 +6275,7 @@ impl<T: BeaconChainTypes> Drop for BeaconChain<T> {
let drop = || -> Result<(), Error> {
self.persist_head_and_fork_choice()?;
self.persist_op_pool()?;
self.persist_data_availabilty_checker()?;
self.persist_eth1_cache()
};

View File

@ -15,6 +15,7 @@ use crate::BeaconChainError;
use eth2::types::BlockContentsTuple;
use kzg::Kzg;
use slog::{debug, warn};
use ssz_derive::{Decode, Encode};
use std::borrow::Cow;
use types::{
BeaconBlockRef, BeaconState, BeaconStateError, BlobSidecar, BlobSidecarList, ChainSpec,
@ -398,8 +399,9 @@ fn cheap_state_advance_to_obtain_committees<'a, E: EthSpec>(
/// Wrapper over a `BlobSidecar` for which we have completed kzg verification.
/// i.e. `verify_blob_kzg_proof(blob, commitment, proof) == true`.
#[derive(Debug, Derivative, Clone)]
#[derive(Debug, Derivative, Clone, Encode, Decode)]
#[derivative(PartialEq, Eq)]
#[ssz(struct_behaviour = "transparent")]
pub struct KzgVerifiedBlob<T: EthSpec> {
blob: Arc<BlobSidecar<T>>,
}

View File

@ -77,6 +77,7 @@ use safe_arith::ArithError;
use slog::{debug, error, warn, Logger};
use slot_clock::SlotClock;
use ssz::Encode;
use ssz_derive::{Decode, Encode};
use state_processing::per_block_processing::{errors::IntoWithIndex, is_merge_transition_block};
use state_processing::{
block_signature_verifier::{BlockSignatureVerifier, Error as BlockSignatureVerifierError},
@ -95,6 +96,7 @@ use task_executor::JoinHandle;
use tree_hash::TreeHash;
use types::blob_sidecar::BlobIdentifier;
use types::ExecPayload;
use types::{ssz_tagged_beacon_state, ssz_tagged_signed_beacon_block};
use types::{
BeaconBlockRef, BeaconState, BeaconStateError, BlindedPayload, ChainSpec, CloneConfig, Epoch,
EthSpec, ExecutionBlockHash, Hash256, InconsistentFork, PublicKey, PublicKeyBytes,
@ -499,7 +501,7 @@ impl<T: EthSpec> From<ArithError> for BlockError<T> {
}
/// Stores information about verifying a payload against an execution engine.
#[derive(Clone)]
#[derive(Debug, PartialEq, Clone, Encode, Decode)]
pub struct PayloadVerificationOutcome {
pub payload_verification_status: PayloadVerificationStatus,
pub is_valid_merge_transition_block: bool,
@ -718,6 +720,7 @@ impl<E: EthSpec> ExecutedBlock<E> {
}
}
#[derive(Debug, PartialEq)]
pub struct AvailableExecutedBlock<E: EthSpec> {
pub block: AvailableBlock<E>,
pub import_data: BlockImportData<E>,
@ -755,6 +758,7 @@ impl<E: EthSpec> AvailableExecutedBlock<E> {
}
}
#[derive(Encode, Decode, Clone)]
pub struct AvailabilityPendingExecutedBlock<E: EthSpec> {
pub block: AvailabilityPendingBlock<E>,
pub import_data: BlockImportData<E>,
@ -799,9 +803,14 @@ impl<E: EthSpec> AvailabilityPendingExecutedBlock<E> {
}
}
#[derive(Debug, PartialEq, Encode, Decode, Clone)]
// TODO (mark): investigate using an Arc<state> / Arc<parent_block>
// here to make this cheaper to clone
pub struct BlockImportData<E: EthSpec> {
pub block_root: Hash256,
#[ssz(with = "ssz_tagged_beacon_state")]
pub state: BeaconState<E>,
#[ssz(with = "ssz_tagged_signed_beacon_block")]
pub parent_block: SignedBeaconBlock<E, BlindedPayload<E>>,
pub parent_eth1_finalization_data: Eth1FinalizationData,
pub confirmed_state_roots: Vec<Hash256>,

View File

@ -794,7 +794,7 @@ where
let beacon_chain = BeaconChain {
spec: self.spec.clone(),
config: self.chain_config,
store,
store: store.clone(),
task_executor: self
.task_executor
.ok_or("Cannot build without task executor")?,
@ -864,8 +864,10 @@ where
data_availability_checker: DataAvailabilityChecker::new(
slot_clock,
kzg.clone(),
store,
self.spec,
),
)
.map_err(|e| format!("Error initializing DataAvailabiltyChecker: {:?}", e))?,
proposal_blob_cache: BlobCache::default(),
kzg,
};

View File

@ -4,23 +4,29 @@ use crate::blob_verification::{
};
use crate::block_verification::{AvailabilityPendingExecutedBlock, AvailableExecutedBlock};
use crate::data_availability_checker::overflow_lru_cache::OverflowLRUCache;
use crate::{BeaconChain, BeaconChainTypes, BeaconStore};
use kzg::Error as KzgError;
use kzg::Kzg;
use parking_lot::RwLock;
use slog::{debug, error};
use slot_clock::SlotClock;
use ssz_types::{Error, FixedVector, VariableList};
use ssz_types::{Error, VariableList};
use state_processing::per_block_processing::deneb::deneb::verify_kzg_commitments_against_transactions;
use std::collections::hash_map::{Entry, OccupiedEntry};
use std::collections::HashMap;
use std::sync::Arc;
use task_executor::TaskExecutor;
use types::beacon_block_body::KzgCommitments;
use types::blob_sidecar::{BlobIdentifier, BlobSidecar};
use types::consts::deneb::MIN_EPOCHS_FOR_BLOB_SIDECARS_REQUESTS;
use types::ssz_tagged_signed_beacon_block;
use types::{
BeaconBlockRef, BlobSidecarList, ChainSpec, Epoch, EthSpec, ExecPayload, FullPayload, Hash256,
SignedBeaconBlock, SignedBeaconBlockHeader, Slot,
};
mod overflow_lru_cache;
pub const OVERFLOW_LRU_CAPACITY: usize = 1024;
#[derive(Debug)]
pub enum AvailabilityCheckError {
DuplicateBlob(Hash256),
@ -39,6 +45,9 @@ pub enum AvailabilityCheckError {
},
Pending,
IncorrectFork,
BlobIndexInvalid(u64),
StoreError(store::Error),
DecodeError(ssz::DecodeError),
}
impl From<ssz_types::Error> for AvailabilityCheckError {
@ -47,70 +56,35 @@ impl From<ssz_types::Error> for AvailabilityCheckError {
}
}
impl From<store::Error> for AvailabilityCheckError {
fn from(value: store::Error) -> Self {
Self::StoreError(value)
}
}
impl From<ssz::DecodeError> for AvailabilityCheckError {
fn from(value: ssz::DecodeError) -> Self {
Self::DecodeError(value)
}
}
/// This cache contains
/// - blobs that have been gossip verified
/// - commitments for blocks that have been gossip verified, but the commitments themselves
/// have not been verified against blobs
/// - blocks that have been fully verified and only require a data availability check
pub struct DataAvailabilityChecker<T: EthSpec, S: SlotClock> {
availability_cache: RwLock<HashMap<Hash256, ReceivedComponents<T>>>,
slot_clock: S,
pub struct DataAvailabilityChecker<T: BeaconChainTypes> {
availability_cache: Arc<OverflowLRUCache<T>>,
slot_clock: T::SlotClock,
kzg: Option<Arc<Kzg>>,
spec: ChainSpec,
}
/// Caches partially available blobs and execution verified blocks corresponding
/// to a given `block_hash` that are received over gossip.
///
/// The blobs are all gossip and kzg verified.
/// The block has completed all verifications except the availability check.
struct ReceivedComponents<T: EthSpec> {
verified_blobs: FixedVector<Option<KzgVerifiedBlob<T>>, T::MaxBlobsPerBlock>,
executed_block: Option<AvailabilityPendingExecutedBlock<T>>,
}
impl<T: EthSpec> ReceivedComponents<T> {
fn new_from_blob(blob: KzgVerifiedBlob<T>) -> Self {
let mut verified_blobs = FixedVector::<_, _>::default();
// TODO: verify that we've already ensured the blob index < T::MaxBlobsPerBlock
if let Some(mut_maybe_blob) = verified_blobs.get_mut(blob.blob_index() as usize) {
*mut_maybe_blob = Some(blob);
}
Self {
verified_blobs,
executed_block: None,
}
}
fn new_from_block(block: AvailabilityPendingExecutedBlock<T>) -> Self {
Self {
verified_blobs: <_>::default(),
executed_block: Some(block),
}
}
/// Returns `true` if the cache has all blobs corresponding to the
/// kzg commitments in the block.
fn has_all_blobs(&self, block: &AvailabilityPendingExecutedBlock<T>) -> bool {
for i in 0..block.num_blobs_expected() {
if self
.verified_blobs
.get(i)
.map(|maybe_blob| maybe_blob.is_none())
.unwrap_or(true)
{
return false;
}
}
true
}
}
/// This type is returned after adding a block / blob to the `DataAvailabilityChecker`.
///
/// Indicates if the block is fully `Available` or if we need blobs or blocks
/// to "complete" the requirements for an `AvailableBlock`.
#[derive(Debug, PartialEq)]
pub enum Availability<T: EthSpec> {
PendingBlobs(Vec<BlobIdentifier>),
PendingBlock(Hash256),
@ -129,25 +103,28 @@ impl<T: EthSpec> Availability<T> {
}
}
impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
pub fn new(slot_clock: S, kzg: Option<Arc<Kzg>>, spec: ChainSpec) -> Self {
Self {
availability_cache: <_>::default(),
impl<T: BeaconChainTypes> DataAvailabilityChecker<T> {
pub fn new(
slot_clock: T::SlotClock,
kzg: Option<Arc<Kzg>>,
store: BeaconStore<T>,
spec: ChainSpec,
) -> Result<Self, AvailabilityCheckError> {
let overflow_cache = OverflowLRUCache::new(OVERFLOW_LRU_CAPACITY, store)?;
Ok(Self {
availability_cache: Arc::new(overflow_cache),
slot_clock,
kzg,
spec,
}
})
}
/// Get a blob from the availability cache.
pub fn get_blob(&self, blob_id: &BlobIdentifier) -> Option<Arc<BlobSidecar<T>>> {
self.availability_cache
.read()
.get(&blob_id.block_root)?
.verified_blobs
.get(blob_id.index as usize)?
.as_ref()
.map(|kzg_verified_blob| kzg_verified_blob.clone_blob())
pub fn get_blob(
&self,
blob_id: &BlobIdentifier,
) -> Result<Option<Arc<BlobSidecar<T::EthSpec>>>, AvailabilityCheckError> {
self.availability_cache.peek_blob(blob_id)
}
/// This first validates the KZG commitments included in the blob sidecar.
@ -158,10 +135,8 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
/// This should only accept gossip verified blobs, so we should not have to worry about dupes.
pub fn put_gossip_blob(
&self,
gossip_blob: GossipVerifiedBlob<T>,
) -> Result<Availability<T>, AvailabilityCheckError> {
let block_root = gossip_blob.block_root();
gossip_blob: GossipVerifiedBlob<T::EthSpec>,
) -> Result<Availability<T::EthSpec>, AvailabilityCheckError> {
// Verify the KZG commitments.
let kzg_verified_blob = if let Some(kzg) = self.kzg.as_ref() {
verify_kzg_for_blob(gossip_blob, kzg)?
@ -169,125 +144,26 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
return Err(AvailabilityCheckError::KzgNotInitialized);
};
let availability = match self
.availability_cache
.write()
.entry(kzg_verified_blob.block_root())
{
Entry::Occupied(mut occupied_entry) => {
// All blobs reaching this cache should be gossip verified and gossip verification
// should filter duplicates, as well as validate indices.
let received_components = occupied_entry.get_mut();
if let Some(maybe_verified_blob) = received_components
.verified_blobs
.get_mut(kzg_verified_blob.blob_index() as usize)
{
*maybe_verified_blob = Some(kzg_verified_blob)
}
if let Some(executed_block) = received_components.executed_block.take() {
self.check_block_availability_maybe_cache(occupied_entry, executed_block)?
} else {
Availability::PendingBlock(block_root)
}
}
Entry::Vacant(vacant_entry) => {
let block_root = kzg_verified_blob.block_root();
vacant_entry.insert(ReceivedComponents::new_from_blob(kzg_verified_blob));
Availability::PendingBlock(block_root)
}
};
Ok(availability)
self.availability_cache
.put_kzg_verified_blob(kzg_verified_blob)
}
/// Check if we have all the blobs for a block. If we do, return the Availability variant that
/// triggers import of the block.
pub fn put_pending_executed_block(
&self,
executed_block: AvailabilityPendingExecutedBlock<T>,
) -> Result<Availability<T>, AvailabilityCheckError> {
let availability = match self
.availability_cache
.write()
.entry(executed_block.import_data.block_root)
{
Entry::Occupied(occupied_entry) => {
self.check_block_availability_maybe_cache(occupied_entry, executed_block)?
}
Entry::Vacant(vacant_entry) => {
let all_blob_ids = executed_block.get_all_blob_ids();
vacant_entry.insert(ReceivedComponents::new_from_block(executed_block));
Availability::PendingBlobs(all_blob_ids)
}
};
Ok(availability)
}
/// Checks if the provided `executed_block` contains all required blobs to be considered an
/// `AvailableBlock` based on blobs that are cached.
///
/// Returns an error if there was an error when matching the block commitments against blob commitments.
///
/// Returns `Ok(Availability::Available(_))` if all blobs for the block are present in cache.
/// Returns `Ok(Availability::PendingBlobs(_))` if all corresponding blobs have not been received in the cache.
fn check_block_availability_maybe_cache(
&self,
mut occupied_entry: OccupiedEntry<Hash256, ReceivedComponents<T>>,
executed_block: AvailabilityPendingExecutedBlock<T>,
) -> Result<Availability<T>, AvailabilityCheckError> {
if occupied_entry.get().has_all_blobs(&executed_block) {
let num_blobs_expected = executed_block.num_blobs_expected();
let AvailabilityPendingExecutedBlock {
block,
import_data,
payload_verification_outcome,
} = executed_block;
let ReceivedComponents {
verified_blobs,
executed_block: _,
} = occupied_entry.remove();
let verified_blobs = Vec::from(verified_blobs)
.into_iter()
.take(num_blobs_expected)
.map(|maybe_blob| maybe_blob.ok_or(AvailabilityCheckError::MissingBlobs))
.collect::<Result<Vec<_>, _>>()?;
let available_block = self.make_available(block, verified_blobs)?;
Ok(Availability::Available(Box::new(
AvailableExecutedBlock::new(
available_block,
import_data,
payload_verification_outcome,
),
)))
} else {
let received_components = occupied_entry.get_mut();
let missing_blob_ids = executed_block.get_filtered_blob_ids(|index| {
received_components
.verified_blobs
.get(index as usize)
.map(|maybe_blob| maybe_blob.is_none())
.unwrap_or(true)
});
let _ = received_components.executed_block.insert(executed_block);
Ok(Availability::PendingBlobs(missing_blob_ids))
}
executed_block: AvailabilityPendingExecutedBlock<T::EthSpec>,
) -> Result<Availability<T::EthSpec>, AvailabilityCheckError> {
self.availability_cache
.put_pending_executed_block(executed_block)
}
/// Checks if a block is available, returns a `MaybeAvailableBlock` that may include the fully
/// available block.
pub fn check_availability(
&self,
block: BlockWrapper<T>,
) -> Result<MaybeAvailableBlock<T>, AvailabilityCheckError> {
block: BlockWrapper<T::EthSpec>,
) -> Result<MaybeAvailableBlock<T::EthSpec>, AvailabilityCheckError> {
match block {
BlockWrapper::Block(block) => self.check_availability_without_blobs(block),
BlockWrapper::BlockAndBlobs(block, blob_list) => {
@ -308,8 +184,8 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
/// Does not access the gossip cache.
pub fn try_check_availability(
&self,
block: BlockWrapper<T>,
) -> Result<AvailableBlock<T>, AvailabilityCheckError> {
block: BlockWrapper<T::EthSpec>,
) -> Result<AvailableBlock<T::EthSpec>, AvailabilityCheckError> {
match block {
BlockWrapper::Block(block) => {
let blob_requirements = self.get_blob_requirements(&block)?;
@ -329,13 +205,13 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
/// commitments are consistent with the provided verified blob commitments.
pub fn check_availability_with_blobs(
&self,
block: Arc<SignedBeaconBlock<T>>,
blobs: KzgVerifiedBlobList<T>,
) -> Result<AvailableBlock<T>, AvailabilityCheckError> {
block: Arc<SignedBeaconBlock<T::EthSpec>>,
blobs: KzgVerifiedBlobList<T::EthSpec>,
) -> Result<AvailableBlock<T::EthSpec>, AvailabilityCheckError> {
match self.check_availability_without_blobs(block)? {
MaybeAvailableBlock::Available(block) => Ok(block),
MaybeAvailableBlock::AvailabilityPending(pending_block) => {
self.make_available(pending_block, blobs)
pending_block.make_available(blobs)
}
}
}
@ -344,8 +220,8 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
/// an AvailableBlock if no blobs are required. Otherwise this will return an AvailabilityPendingBlock.
pub fn check_availability_without_blobs(
&self,
block: Arc<SignedBeaconBlock<T>>,
) -> Result<MaybeAvailableBlock<T>, AvailabilityCheckError> {
block: Arc<SignedBeaconBlock<T::EthSpec>>,
) -> Result<MaybeAvailableBlock<T::EthSpec>, AvailabilityCheckError> {
let blob_requirements = self.get_blob_requirements(&block)?;
let blobs = match blob_requirements {
BlobRequirements::EmptyBlobs => VerifiedBlobs::EmptyBlobs,
@ -363,50 +239,18 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
}))
}
/// Verifies an AvailabilityPendingBlock against a set of KZG verified blobs.
/// This does not check whether a block *should* have blobs, these checks should must have been
/// completed when producing the `AvailabilityPendingBlock`.
pub fn make_available(
&self,
block: AvailabilityPendingBlock<T>,
blobs: Vec<KzgVerifiedBlob<T>>,
) -> Result<AvailableBlock<T>, AvailabilityCheckError> {
let block_kzg_commitments = block.kzg_commitments()?;
if blobs.len() != block_kzg_commitments.len() {
return Err(AvailabilityCheckError::NumBlobsMismatch {
num_kzg_commitments: block_kzg_commitments.len(),
num_blobs: blobs.len(),
});
}
for (block_commitment, blob) in block_kzg_commitments.iter().zip(blobs.iter()) {
if *block_commitment != blob.kzg_commitment() {
return Err(AvailabilityCheckError::KzgCommitmentMismatch {
blob_index: blob.as_blob().index,
});
}
}
let blobs = VariableList::new(blobs.into_iter().map(|blob| blob.to_blob()).collect())?;
Ok(AvailableBlock {
block: block.block,
blobs: VerifiedBlobs::Available(blobs),
})
}
/// Determines the blob requirements for a block. Answers the question: "Does this block require
/// blobs?".
fn get_blob_requirements(
&self,
block: &Arc<SignedBeaconBlock<T, FullPayload<T>>>,
block: &Arc<SignedBeaconBlock<T::EthSpec, FullPayload<T::EthSpec>>>,
) -> Result<BlobRequirements, AvailabilityCheckError> {
let verified_blobs = if let (Ok(block_kzg_commitments), Ok(payload)) = (
block.message().body().blob_kzg_commitments(),
block.message().body().execution_payload(),
) {
if let Some(transactions) = payload.transactions() {
let verified = verify_kzg_commitments_against_transactions::<T>(
let verified = verify_kzg_commitments_against_transactions::<T::EthSpec>(
transactions,
block_kzg_commitments,
)
@ -437,7 +281,7 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
self.spec.deneb_fork_epoch.and_then(|fork_epoch| {
self.slot_clock
.now()
.map(|slot| slot.epoch(T::slots_per_epoch()))
.map(|slot| slot.epoch(T::EthSpec::slots_per_epoch()))
.map(|current_epoch| {
std::cmp::max(
fork_epoch,
@ -452,6 +296,96 @@ impl<T: EthSpec, S: SlotClock> DataAvailabilityChecker<T, S> {
self.data_availability_boundary()
.map_or(false, |da_epoch| block_epoch >= da_epoch)
}
/// Persist all in memory components to disk
pub fn persist_all(&self) -> Result<(), AvailabilityCheckError> {
self.availability_cache.write_all_to_disk()
}
}
pub fn start_availability_cache_maintenance_service<T: BeaconChainTypes>(
executor: TaskExecutor,
chain: Arc<BeaconChain<T>>,
) {
// this cache only needs to be maintained if deneb is configured
if chain.spec.deneb_fork_epoch.is_some() {
let overflow_cache = chain.data_availability_checker.availability_cache.clone();
executor.spawn(
async move { availability_cache_maintenance_service(chain, overflow_cache).await },
"availability_cache_service",
);
} else {
debug!(
chain.log,
"Deneb fork not configured, not starting availability cache maintenance service"
);
}
}
async fn availability_cache_maintenance_service<T: BeaconChainTypes>(
chain: Arc<BeaconChain<T>>,
overflow_cache: Arc<OverflowLRUCache<T>>,
) {
let epoch_duration = chain.slot_clock.slot_duration() * T::EthSpec::slots_per_epoch() as u32;
loop {
match chain
.slot_clock
.duration_to_next_epoch(T::EthSpec::slots_per_epoch())
{
Some(duration) => {
// this service should run 3/4 of the way through the epoch
let additional_delay = (epoch_duration * 3) / 4;
tokio::time::sleep(duration + additional_delay).await;
let deneb_fork_epoch = match chain.spec.deneb_fork_epoch {
Some(epoch) => epoch,
None => break, // shutdown service if deneb fork epoch not set
};
debug!(
chain.log,
"Availability cache maintenance service firing";
);
let current_epoch = match chain
.slot_clock
.now()
.map(|slot| slot.epoch(T::EthSpec::slots_per_epoch()))
{
Some(epoch) => epoch,
None => continue, // we'll have to try again next time I suppose..
};
if current_epoch < deneb_fork_epoch {
// we are not in deneb yet
continue;
}
let finalized_epoch = chain
.canonical_head
.fork_choice_read_lock()
.finalized_checkpoint()
.epoch;
// any data belonging to an epoch before this should be pruned
let cutoff_epoch = std::cmp::max(
finalized_epoch + 1,
std::cmp::max(
current_epoch.saturating_sub(*MIN_EPOCHS_FOR_BLOB_SIDECARS_REQUESTS),
deneb_fork_epoch,
),
);
if let Err(e) = overflow_cache.do_maintenance(cutoff_epoch) {
error!(chain.log, "Failed to maintain availability cache"; "error" => ?e);
}
}
None => {
error!(chain.log, "Failed to read slot clock");
// If we can't read the slot clock, just wait another slot.
tokio::time::sleep(chain.slot_clock.slot_duration()).await;
}
};
}
}
pub enum BlobRequirements {
@ -493,6 +427,37 @@ impl<E: EthSpec> AvailabilityPendingBlock<E> {
.blob_kzg_commitments()
.map_err(|_| AvailabilityCheckError::IncorrectFork)
}
/// Verifies an AvailabilityPendingBlock against a set of KZG verified blobs.
/// This does not check whether a block *should* have blobs, these checks should must have been
/// completed when producing the `AvailabilityPendingBlock`.
pub fn make_available(
self,
blobs: Vec<KzgVerifiedBlob<E>>,
) -> Result<AvailableBlock<E>, AvailabilityCheckError> {
let block_kzg_commitments = self.kzg_commitments()?;
if blobs.len() != block_kzg_commitments.len() {
return Err(AvailabilityCheckError::NumBlobsMismatch {
num_kzg_commitments: block_kzg_commitments.len(),
num_blobs: blobs.len(),
});
}
for (block_commitment, blob) in block_kzg_commitments.iter().zip(blobs.iter()) {
if *block_commitment != blob.kzg_commitment() {
return Err(AvailabilityCheckError::KzgCommitmentMismatch {
blob_index: blob.as_blob().index,
});
}
}
let blobs = VariableList::new(blobs.into_iter().map(|blob| blob.to_blob()).collect())?;
Ok(AvailableBlock {
block: self.block,
blobs: VerifiedBlobs::Available(blobs),
})
}
}
#[derive(Clone, Debug, PartialEq)]
@ -576,3 +541,44 @@ impl<E: EthSpec> AsBlock<E> for AvailableBlock<E> {
}
}
}
// The standard implementation of Encode for SignedBeaconBlock
// requires us to use ssz(enum_behaviour = "transparent"). This
// prevents us from implementing Decode. We need to use a
// custom Encode and Decode in this wrapper object that essentially
// encodes it as if it were ssz(enum_behaviour = "union")
impl<E: EthSpec> ssz::Encode for AvailabilityPendingBlock<E> {
fn is_ssz_fixed_len() -> bool {
ssz_tagged_signed_beacon_block::encode::is_ssz_fixed_len()
}
fn ssz_append(&self, buf: &mut Vec<u8>) {
ssz_tagged_signed_beacon_block::encode::ssz_append(self.block.as_ref(), buf);
}
fn ssz_bytes_len(&self) -> usize {
ssz_tagged_signed_beacon_block::encode::ssz_bytes_len(self.block.as_ref())
}
}
impl<E: EthSpec> ssz::Decode for AvailabilityPendingBlock<E> {
fn is_ssz_fixed_len() -> bool {
ssz_tagged_signed_beacon_block::decode::is_ssz_fixed_len()
}
fn from_ssz_bytes(bytes: &[u8]) -> Result<Self, ssz::DecodeError> {
Ok(Self {
block: Arc::new(ssz_tagged_signed_beacon_block::decode::from_ssz_bytes(
bytes,
)?),
})
}
}
#[cfg(test)]
mod test {
#[test]
fn check_encode_decode_availability_pending_block() {
// todo.. (difficult to create default beacon blocks to test)
}
}

File diff suppressed because it is too large Load Diff

View File

@ -2,6 +2,7 @@ use crate::attester_cache::Error as AttesterCacheError;
use crate::beacon_block_streamer::Error as BlockStreamerError;
use crate::beacon_chain::ForkChoiceError;
use crate::beacon_fork_choice_store::Error as ForkChoiceStoreError;
use crate::data_availability_checker::AvailabilityCheckError;
use crate::eth1_chain::Error as Eth1ChainError;
use crate::historical_blocks::HistoricalBlockError;
use crate::migrate::PruningError;
@ -215,6 +216,7 @@ pub enum BeaconChainError {
BlsToExecutionConflictsWithPool,
InconsistentFork(InconsistentFork),
ProposerHeadForkChoiceError(fork_choice::Error<proto_array::Error>),
AvailabilityCheckError(AvailabilityCheckError),
}
easy_from_to!(SlotProcessingError, BeaconChainError);
@ -240,6 +242,7 @@ easy_from_to!(HistoricalBlockError, BeaconChainError);
easy_from_to!(StateAdvanceError, BeaconChainError);
easy_from_to!(BlockReplayError, BeaconChainError);
easy_from_to!(InconsistentFork, BeaconChainError);
easy_from_to!(AvailabilityCheckError, BeaconChainError);
#[derive(Debug)]
pub enum BlockProductionError {

View File

@ -1,4 +1,5 @@
use slog::{debug, Logger};
use ssz_derive::{Decode, Encode};
use std::cmp;
use std::collections::BTreeMap;
use types::{Checkpoint, Epoch, Eth1Data, Hash256 as Root};
@ -10,7 +11,7 @@ pub const DEFAULT_ETH1_CACHE_SIZE: usize = 5;
/// These fields are named the same as the corresponding fields in the `BeaconState`
/// as this structure stores these values from the `BeaconState` at a `Checkpoint`
#[derive(Clone)]
#[derive(Clone, Debug, PartialEq, Encode, Decode)]
pub struct Eth1FinalizationData {
pub eth1_data: Eth1Data,
pub eth1_deposit_index: u64,

View File

@ -380,6 +380,8 @@ lazy_static! {
try_create_histogram("beacon_persist_eth1_cache", "Time taken to persist the eth1 caches");
pub static ref PERSIST_FORK_CHOICE: Result<Histogram> =
try_create_histogram("beacon_persist_fork_choice", "Time taken to persist the fork choice struct");
pub static ref PERSIST_DATA_AVAILABILITY_CHECKER: Result<Histogram> =
try_create_histogram("beacon_persist_data_availability_checker", "Time taken to persist the data availability checker");
/*
* Eth1

View File

@ -2,6 +2,7 @@ use crate::address_change_broadcast::broadcast_address_changes_at_capella;
use crate::config::{ClientGenesis, Config as ClientConfig};
use crate::notifier::spawn_notifier;
use crate::Client;
use beacon_chain::data_availability_checker::start_availability_cache_maintenance_service;
use beacon_chain::otb_verification_service::start_otb_verification_service;
use beacon_chain::proposer_prep_service::start_proposer_prep_service;
use beacon_chain::schema_change::migrate_schema;
@ -828,6 +829,10 @@ where
start_proposer_prep_service(runtime_context.executor.clone(), beacon_chain.clone());
start_otb_verification_service(runtime_context.executor.clone(), beacon_chain.clone());
start_availability_cache_maintenance_service(
runtime_context.executor.clone(),
beacon_chain.clone(),
);
}
Ok(Client {

View File

@ -16,6 +16,7 @@ use std::collections::HashMap;
use std::sync::Arc;
use tree_hash::TreeHash;
use tree_hash_derive::TreeHash;
use types::consts::deneb::BLOB_TX_TYPE;
use types::transaction::{BlobTransaction, EcdsaSignature, SignedBlobTransaction};
use types::{
Blob, EthSpec, ExecutionBlockHash, ExecutionPayload, ExecutionPayloadCapella,
@ -684,7 +685,7 @@ impl<T: EthSpec> ExecutionBlockGenerator<T> {
signature: bad_signature,
};
// calculate transaction bytes
let tx_bytes = [0x05u8]
let tx_bytes = [BLOB_TX_TYPE]
.into_iter()
.chain(signed_blob_transaction.as_ssz_bytes().into_iter())
.collect::<Vec<_>>();

View File

@ -230,7 +230,7 @@ impl<T: BeaconChainTypes> Worker<T> {
let mut blob_list_results = HashMap::new();
for id in request.blob_ids.into_iter() {
// First attempt to get the blobs from the RPC cache.
if let Some(blob) = self.chain.data_availability_checker.get_blob(&id) {
if let Ok(Some(blob)) = self.chain.data_availability_checker.get_blob(&id) {
self.send_response(peer_id, Response::BlobsByRoot(Some(blob)), request_id);
send_blob_count += 1;
} else {

View File

@ -198,6 +198,36 @@ impl<E: EthSpec> KeyValueStore<E> for LevelDB<E> {
)
}
fn iter_raw_entries(&self, column: DBColumn, prefix: &[u8]) -> RawEntryIter {
let start_key = BytesKey::from_vec(get_key_for_col(column.into(), prefix));
let iter = self.db.iter(self.read_options());
iter.seek(&start_key);
Box::new(
iter.take_while(move |(key, _)| key.key.starts_with(start_key.key.as_slice()))
.map(move |(bytes_key, value)| {
let subkey = &bytes_key.key[column.as_bytes().len()..];
Ok((Vec::from(subkey), value))
}),
)
}
fn iter_raw_keys(&self, column: DBColumn, prefix: &[u8]) -> RawKeyIter {
let start_key = BytesKey::from_vec(get_key_for_col(column.into(), prefix));
let iter = self.db.keys_iter(self.read_options());
iter.seek(&start_key);
Box::new(
iter.take_while(move |key| key.key.starts_with(start_key.key.as_slice()))
.map(move |bytes_key| {
let subkey = &bytes_key.key[column.as_bytes().len()..];
Ok(Vec::from(subkey))
}),
)
}
/// Iterate through all keys and values in a particular column.
fn iter_column_keys(&self, column: DBColumn) -> ColumnKeyIter {
let start_key =

View File

@ -49,6 +49,9 @@ pub use types::*;
pub type ColumnIter<'a> = Box<dyn Iterator<Item = Result<(Hash256, Vec<u8>), Error>> + 'a>;
pub type ColumnKeyIter<'a> = Box<dyn Iterator<Item = Result<Hash256, Error>> + 'a>;
pub type RawEntryIter<'a> = Box<dyn Iterator<Item = Result<(Vec<u8>, Vec<u8>), Error>> + 'a>;
pub type RawKeyIter<'a> = Box<dyn Iterator<Item = Result<Vec<u8>, Error>> + 'a>;
pub trait KeyValueStore<E: EthSpec>: Sync + Send + Sized + 'static {
/// Retrieve some bytes in `column` with `key`.
fn get_bytes(&self, column: &str, key: &[u8]) -> Result<Option<Vec<u8>>, Error>;
@ -88,6 +91,14 @@ pub trait KeyValueStore<E: EthSpec>: Sync + Send + Sized + 'static {
Box::new(std::iter::empty())
}
fn iter_raw_entries(&self, _column: DBColumn, _prefix: &[u8]) -> RawEntryIter {
Box::new(std::iter::empty())
}
fn iter_raw_keys(&self, _column: DBColumn, _prefix: &[u8]) -> RawKeyIter {
Box::new(std::iter::empty())
}
/// Iterate through all keys in a particular column.
fn iter_column_keys(&self, _column: DBColumn) -> ColumnKeyIter {
// Default impl for non LevelDB databases
@ -227,6 +238,8 @@ pub enum DBColumn {
OptimisticTransitionBlock,
#[strum(serialize = "bhs")]
BeaconHistoricalSummaries,
#[strum(serialize = "olc")]
OverflowLRUCache,
}
/// A block from the database, which might have an execution payload or not.

View File

@ -192,7 +192,8 @@ impl CountUnrealized {
/// Indicates if a block has been verified by an execution payload.
///
/// There is no variant for "invalid", since such a block should never be added to fork choice.
#[derive(Clone, Copy, Debug, PartialEq)]
#[derive(Clone, Copy, Debug, PartialEq, Encode, Decode)]
#[ssz(enum_behaviour = "tag")]
pub enum PayloadVerificationStatus {
/// An EL has declared the execution payload to be valid.
Verified,

View File

@ -4,6 +4,7 @@
//!
//! The following struct/enum attributes are available:
//!
//! - `#[ssz(enum_behaviour = "tag")]`: encodes and decodes an `enum` with 0 fields per variant
//! - `#[ssz(enum_behaviour = "union")]`: encodes and decodes an `enum` with a one-byte variant selector.
//! - `#[ssz(enum_behaviour = "transparent")]`: allows encoding an `enum` by serializing only the
//! value whilst ignoring outermost the `enum`.
@ -140,6 +141,22 @@
//! TransparentEnum::Bar(vec![42, 42]).as_ssz_bytes(),
//! vec![42, 42]
//! );
//!
//! /// Representated as an SSZ "uint8"
//! #[derive(Debug, PartialEq, Encode, Decode)]
//! #[ssz(enum_behaviour = "tag")]
//! enum TagEnum {
//! Foo,
//! Bar,
//! }
//! assert_eq!(
//! TagEnum::Foo.as_ssz_bytes(),
//! vec![0]
//! );
//! assert_eq!(
//! TagEnum::from_ssz_bytes(&[1]).unwrap(),
//! TagEnum::Bar,
//! );
//! ```
use darling::{FromDeriveInput, FromMeta};
@ -154,8 +171,9 @@ const MAX_UNION_SELECTOR: u8 = 127;
const ENUM_TRANSPARENT: &str = "transparent";
const ENUM_UNION: &str = "union";
const ENUM_TAG: &str = "tag";
const NO_ENUM_BEHAVIOUR_ERROR: &str = "enums require an \"enum_behaviour\" attribute with \
a \"transparent\" or \"union\" value, e.g., #[ssz(enum_behaviour = \"transparent\")]";
a \"transparent\", \"union\", or \"tag\" value, e.g., #[ssz(enum_behaviour = \"transparent\")]";
#[derive(Debug, FromDeriveInput)]
#[darling(attributes(ssz))]
@ -196,6 +214,7 @@ enum StructBehaviour {
enum EnumBehaviour {
Union,
Transparent,
Tag,
}
impl<'a> Procedure<'a> {
@ -237,6 +256,10 @@ impl<'a> Procedure<'a> {
data,
behaviour: EnumBehaviour::Transparent,
},
Some("tag") => Procedure::Enum {
data,
behaviour: EnumBehaviour::Tag,
},
Some(other) => panic!(
"{} is not a valid enum behaviour, use \"container\" or \"transparent\"",
other
@ -296,6 +319,7 @@ pub fn ssz_encode_derive(input: TokenStream) -> TokenStream {
Procedure::Enum { data, behaviour } => match behaviour {
EnumBehaviour::Transparent => ssz_encode_derive_enum_transparent(&item, data),
EnumBehaviour::Union => ssz_encode_derive_enum_union(&item, data),
EnumBehaviour::Tag => ssz_encode_derive_enum_tag(&item, data),
},
}
}
@ -573,6 +597,67 @@ fn ssz_encode_derive_enum_transparent(
output.into()
}
/// Derive `ssz::Encode` for an `enum` following the "tag" method.
///
/// The union selector will be determined based upon the order in which the enum variants are
/// defined. E.g., the top-most variant in the enum will have a selector of `0`, the variant
/// beneath it will have a selector of `1` and so on.
///
/// # Limitations
///
/// Only supports enums where each variant has no fields
fn ssz_encode_derive_enum_tag(derive_input: &DeriveInput, enum_data: &DataEnum) -> TokenStream {
let name = &derive_input.ident;
let (impl_generics, ty_generics, where_clause) = &derive_input.generics.split_for_impl();
let patterns: Vec<_> = enum_data
.variants
.iter()
.map(|variant| {
let variant_name = &variant.ident;
if !variant.fields.is_empty() {
panic!("ssz::Encode tag behaviour can only be derived for enums with no fields");
}
quote! {
#name::#variant_name
}
})
.collect();
let union_selectors = compute_union_selectors(patterns.len());
let output = quote! {
impl #impl_generics ssz::Encode for #name #ty_generics #where_clause {
fn is_ssz_fixed_len() -> bool {
true
}
fn ssz_fixed_len() -> usize {
1
}
fn ssz_bytes_len(&self) -> usize {
1
}
fn ssz_append(&self, buf: &mut Vec<u8>) {
match self {
#(
#patterns => {
let union_selector: u8 = #union_selectors;
debug_assert!(union_selector <= ssz::MAX_UNION_SELECTOR);
buf.push(union_selector);
},
)*
}
}
}
};
output.into()
}
/// Derive `ssz::Encode` for an `enum` following the "union" SSZ spec.
///
/// The union selector will be determined based upon the order in which the enum variants are
@ -652,9 +737,10 @@ pub fn ssz_decode_derive(input: TokenStream) -> TokenStream {
},
Procedure::Enum { data, behaviour } => match behaviour {
EnumBehaviour::Union => ssz_decode_derive_enum_union(&item, data),
EnumBehaviour::Tag => ssz_decode_derive_enum_tag(&item, data),
EnumBehaviour::Transparent => panic!(
"Decode cannot be derived for enum_behaviour \"{}\", only \"{}\" is valid.",
ENUM_TRANSPARENT, ENUM_UNION
"Decode cannot be derived for enum_behaviour \"{}\", only \"{}\" and \"{}\" is valid.",
ENUM_TRANSPARENT, ENUM_UNION, ENUM_TAG,
),
},
}
@ -908,6 +994,59 @@ fn ssz_decode_derive_struct_transparent(
output.into()
}
/// Derive `ssz::Decode` for an `enum` following the "tag" SSZ spec.
fn ssz_decode_derive_enum_tag(derive_input: &DeriveInput, enum_data: &DataEnum) -> TokenStream {
let name = &derive_input.ident;
let (impl_generics, ty_generics, where_clause) = &derive_input.generics.split_for_impl();
let patterns: Vec<_> = enum_data
.variants
.iter()
.map(|variant| {
let variant_name = &variant.ident;
if !variant.fields.is_empty() {
panic!("ssz::Decode tag behaviour can only be derived for enums with no fields");
}
quote! {
#name::#variant_name
}
})
.collect();
let union_selectors = compute_union_selectors(patterns.len());
let output = quote! {
impl #impl_generics ssz::Decode for #name #ty_generics #where_clause {
fn is_ssz_fixed_len() -> bool {
true
}
fn ssz_fixed_len() -> usize {
1
}
fn from_ssz_bytes(bytes: &[u8]) -> std::result::Result<Self, ssz::DecodeError> {
let byte = bytes
.first()
.copied()
.ok_or(ssz::DecodeError::OutOfBoundsByte { i: 0 })?;
match byte {
#(
#union_selectors => {
Ok(#patterns)
},
)*
other => Err(ssz::DecodeError::UnionSelectorInvalid(other)),
}
}
}
};
output.into()
}
/// Derive `ssz::Decode` for an `enum` following the "union" SSZ spec.
fn ssz_decode_derive_enum_union(derive_input: &DeriveInput, enum_data: &DataEnum) -> TokenStream {
let name = &derive_input.ident;

View File

@ -12,6 +12,14 @@ fn assert_encode_decode<T: Encode + Decode + PartialEq + Debug>(item: &T, bytes:
assert_eq!(T::from_ssz_bytes(bytes).unwrap(), *item);
}
#[derive(PartialEq, Debug, Encode, Decode)]
#[ssz(enum_behaviour = "tag")]
enum TagEnum {
A,
B,
C,
}
#[derive(PartialEq, Debug, Encode, Decode)]
#[ssz(enum_behaviour = "union")]
enum TwoFixedUnion {
@ -120,6 +128,13 @@ fn two_variable_union() {
);
}
#[test]
fn tag_enum() {
assert_encode_decode(&TagEnum::A, &[0]);
assert_encode_decode(&TagEnum::B, &[1]);
assert_encode_decode(&TagEnum::C, &[2]);
}
#[derive(PartialEq, Debug, Encode, Decode)]
#[ssz(enum_behaviour = "union")]
enum TwoVecUnion {

View File

@ -1,5 +1,6 @@
use crate::common::get_indexed_attestation;
use crate::per_block_processing::errors::{AttestationInvalid, BlockOperationError};
use ssz_derive::{Decode, Encode};
use std::collections::{hash_map::Entry, HashMap};
use tree_hash::TreeHash;
use types::{
@ -7,7 +8,7 @@ use types::{
ChainSpec, Epoch, EthSpec, Hash256, IndexedAttestation, SignedBeaconBlock, Slot,
};
#[derive(Debug, Clone)]
#[derive(Debug, PartialEq, Clone, Encode, Decode)]
pub struct ConsensusContext<T: EthSpec> {
/// Slot to act as an identifier/safeguard
slot: Slot,
@ -16,6 +17,8 @@ pub struct ConsensusContext<T: EthSpec> {
/// Block root of the block at `slot`.
current_block_root: Option<Hash256>,
/// Cache of indexed attestations constructed during block processing.
/// We can skip serializing / deserializing this as the cache will just be rebuilt
#[ssz(skip_serializing, skip_deserializing)]
indexed_attestations:
HashMap<(AttestationData, BitList<T::MaxValidatorsPerCommittee>), IndexedAttestation<T>>,
/// Whether `verify_kzg_commitments_against_transactions` has successfully passed.

View File

@ -201,13 +201,7 @@ impl<'a, T: EthSpec, Payload: AbstractExecPayload<T>> BeaconBlockRef<'a, T, Payl
/// dictated by `self.slot()`.
pub fn fork_name(&self, spec: &ChainSpec) -> Result<ForkName, InconsistentFork> {
let fork_at_slot = spec.fork_name_at_slot::<T>(self.slot());
let object_fork = match self {
BeaconBlockRef::Base { .. } => ForkName::Base,
BeaconBlockRef::Altair { .. } => ForkName::Altair,
BeaconBlockRef::Merge { .. } => ForkName::Merge,
BeaconBlockRef::Capella { .. } => ForkName::Capella,
BeaconBlockRef::Deneb { .. } => ForkName::Deneb,
};
let object_fork = self.fork_name_unchecked();
if fork_at_slot == object_fork {
Ok(object_fork)
@ -219,6 +213,19 @@ impl<'a, T: EthSpec, Payload: AbstractExecPayload<T>> BeaconBlockRef<'a, T, Payl
}
}
/// Returns the name of the fork pertaining to `self`.
///
/// Does not check that the fork is consistent with the slot.
pub fn fork_name_unchecked(&self) -> ForkName {
match self {
BeaconBlockRef::Base { .. } => ForkName::Base,
BeaconBlockRef::Altair { .. } => ForkName::Altair,
BeaconBlockRef::Merge { .. } => ForkName::Merge,
BeaconBlockRef::Capella { .. } => ForkName::Capella,
BeaconBlockRef::Deneb { .. } => ForkName::Deneb,
}
}
/// Convenience accessor for the `body` as a `BeaconBlockBodyRef`.
pub fn body(&self) -> BeaconBlockBodyRef<'a, T, Payload> {
map_beacon_block_ref_into_beacon_block_body_ref!(&'a _, *self, |block, cons| cons(

View File

@ -415,13 +415,7 @@ impl<T: EthSpec> BeaconState<T> {
/// dictated by `self.slot()`.
pub fn fork_name(&self, spec: &ChainSpec) -> Result<ForkName, InconsistentFork> {
let fork_at_slot = spec.fork_name_at_epoch(self.current_epoch());
let object_fork = match self {
BeaconState::Base { .. } => ForkName::Base,
BeaconState::Altair { .. } => ForkName::Altair,
BeaconState::Merge { .. } => ForkName::Merge,
BeaconState::Capella { .. } => ForkName::Capella,
BeaconState::Deneb { .. } => ForkName::Deneb,
};
let object_fork = self.fork_name_unchecked();
if fork_at_slot == object_fork {
Ok(object_fork)
@ -433,6 +427,19 @@ impl<T: EthSpec> BeaconState<T> {
}
}
/// Returns the name of the fork pertaining to `self`.
///
/// Does not check if `self` is consistent with the fork dictated by `self.slot()`.
pub fn fork_name_unchecked(&self) -> ForkName {
match self {
BeaconState::Base { .. } => ForkName::Base,
BeaconState::Altair { .. } => ForkName::Altair,
BeaconState::Merge { .. } => ForkName::Merge,
BeaconState::Capella { .. } => ForkName::Capella,
BeaconState::Deneb { .. } => ForkName::Deneb,
}
}
/// Specialised deserialisation method that uses the `ChainSpec` as context.
#[allow(clippy::integer_arithmetic)]
pub fn from_ssz_bytes(bytes: &[u8], spec: &ChainSpec) -> Result<Self, ssz::DecodeError> {
@ -1870,3 +1877,80 @@ impl<T: EthSpec> ForkVersionDeserialize for BeaconState<T> {
))
}
}
/// This module can be used to encode and decode a `BeaconState` the same way it
/// would be done if we had tagged the superstruct enum with
/// `#[ssz(enum_behaviour = "union")]`
/// This should _only_ be used for *some* cases to store these objects in the
/// database and _NEVER_ for encoding / decoding states sent over the network!
pub mod ssz_tagged_beacon_state {
use super::*;
pub mod encode {
use super::*;
#[allow(unused_imports)]
use ssz::*;
pub fn is_ssz_fixed_len() -> bool {
false
}
pub fn ssz_fixed_len() -> usize {
BYTES_PER_LENGTH_OFFSET
}
pub fn ssz_bytes_len<E: EthSpec>(state: &BeaconState<E>) -> usize {
state
.ssz_bytes_len()
.checked_add(1)
.expect("encoded length must be less than usize::max")
}
pub fn ssz_append<E: EthSpec>(state: &BeaconState<E>, buf: &mut Vec<u8>) {
let fork_name = state.fork_name_unchecked();
fork_name.ssz_append(buf);
state.ssz_append(buf);
}
pub fn as_ssz_bytes<E: EthSpec>(state: &BeaconState<E>) -> Vec<u8> {
let mut buf = vec![];
ssz_append(state, &mut buf);
buf
}
}
pub mod decode {
use super::*;
#[allow(unused_imports)]
use ssz::*;
pub fn is_ssz_fixed_len() -> bool {
false
}
pub fn ssz_fixed_len() -> usize {
BYTES_PER_LENGTH_OFFSET
}
pub fn from_ssz_bytes<E: EthSpec>(bytes: &[u8]) -> Result<BeaconState<E>, DecodeError> {
let fork_byte = bytes
.first()
.copied()
.ok_or(DecodeError::OutOfBoundsByte { i: 0 })?;
let body = bytes
.get(1..)
.ok_or(DecodeError::OutOfBoundsByte { i: 1 })?;
match ForkName::from_ssz_bytes(&[fork_byte])? {
ForkName::Base => Ok(BeaconState::Base(BeaconStateBase::from_ssz_bytes(body)?)),
ForkName::Altair => Ok(BeaconState::Altair(BeaconStateAltair::from_ssz_bytes(
body,
)?)),
ForkName::Merge => Ok(BeaconState::Merge(BeaconStateMerge::from_ssz_bytes(body)?)),
ForkName::Capella => Ok(BeaconState::Capella(BeaconStateCapella::from_ssz_bytes(
body,
)?)),
ForkName::Deneb => Ok(BeaconState::Deneb(BeaconStateDeneb::from_ssz_bytes(body)?)),
}
}
}
}

View File

@ -1,12 +1,14 @@
use crate::{ChainSpec, Epoch};
use serde_derive::{Deserialize, Serialize};
use ssz_derive::{Decode, Encode};
use std::convert::TryFrom;
use std::fmt::{self, Display, Formatter};
use std::str::FromStr;
#[derive(Debug, Clone, Copy, PartialEq, Eq, Hash, Serialize, Deserialize)]
#[derive(Debug, Clone, Copy, Decode, Encode, PartialEq, Eq, Hash, Serialize, Deserialize)]
#[serde(try_from = "String")]
#[serde(into = "String")]
#[ssz(enum_behaviour = "tag")]
pub enum ForkName {
Base,
Altair,

View File

@ -170,9 +170,9 @@ pub use crate::selection_proof::SelectionProof;
pub use crate::shuffling_id::AttestationShufflingId;
pub use crate::signed_aggregate_and_proof::SignedAggregateAndProof;
pub use crate::signed_beacon_block::{
SignedBeaconBlock, SignedBeaconBlockAltair, SignedBeaconBlockBase, SignedBeaconBlockCapella,
SignedBeaconBlockDeneb, SignedBeaconBlockHash, SignedBeaconBlockMerge,
SignedBlindedBeaconBlock,
ssz_tagged_signed_beacon_block, SignedBeaconBlock, SignedBeaconBlockAltair,
SignedBeaconBlockBase, SignedBeaconBlockCapella, SignedBeaconBlockDeneb, SignedBeaconBlockHash,
SignedBeaconBlockMerge, SignedBlindedBeaconBlock,
};
pub use crate::signed_beacon_block_header::SignedBeaconBlockHeader;
pub use crate::signed_blob::*;

View File

@ -92,6 +92,12 @@ impl<E: EthSpec, Payload: AbstractExecPayload<E>> SignedBeaconBlock<E, Payload>
self.message().fork_name(spec)
}
/// Returns the name of the fork pertaining to `self`
/// Does not check that the fork is consistent with the slot.
pub fn fork_name_unchecked(&self) -> ForkName {
self.message().fork_name_unchecked()
}
/// SSZ decode with fork variant determined by slot.
pub fn from_ssz_bytes(bytes: &[u8], spec: &ChainSpec) -> Result<Self, ssz::DecodeError> {
Self::from_ssz_bytes_with(bytes, |bytes| BeaconBlock::from_ssz_bytes(bytes, spec))
@ -510,6 +516,99 @@ impl<E: EthSpec, Payload: AbstractExecPayload<E>> ForkVersionDeserialize
}
}
/// This module can be used to encode and decode a `SignedBeaconBlock` the same way it
/// would be done if we had tagged the superstruct enum with
/// `#[ssz(enum_behaviour = "union")]`
/// This should _only_ be used *some* cases when storing these objects in the database
/// and _NEVER_ for encoding / decoding blocks sent over the network!
pub mod ssz_tagged_signed_beacon_block {
use super::*;
pub mod encode {
use super::*;
#[allow(unused_imports)]
use ssz::*;
pub fn is_ssz_fixed_len() -> bool {
false
}
pub fn ssz_fixed_len() -> usize {
BYTES_PER_LENGTH_OFFSET
}
pub fn ssz_bytes_len<E: EthSpec, Payload: AbstractExecPayload<E>>(
block: &SignedBeaconBlock<E, Payload>,
) -> usize {
block
.ssz_bytes_len()
.checked_add(1)
.expect("encoded length must be less than usize::max")
}
pub fn ssz_append<E: EthSpec, Payload: AbstractExecPayload<E>>(
block: &SignedBeaconBlock<E, Payload>,
buf: &mut Vec<u8>,
) {
let fork_name = block.fork_name_unchecked();
fork_name.ssz_append(buf);
block.ssz_append(buf);
}
pub fn as_ssz_bytes<E: EthSpec, Payload: AbstractExecPayload<E>>(
block: &SignedBeaconBlock<E, Payload>,
) -> Vec<u8> {
let mut buf = vec![];
ssz_append(block, &mut buf);
buf
}
}
pub mod decode {
use super::*;
#[allow(unused_imports)]
use ssz::*;
pub fn is_ssz_fixed_len() -> bool {
false
}
pub fn ssz_fixed_len() -> usize {
BYTES_PER_LENGTH_OFFSET
}
pub fn from_ssz_bytes<E: EthSpec, Payload: AbstractExecPayload<E>>(
bytes: &[u8],
) -> Result<SignedBeaconBlock<E, Payload>, DecodeError> {
let fork_byte = bytes
.first()
.copied()
.ok_or(DecodeError::OutOfBoundsByte { i: 0 })?;
let body = bytes
.get(1..)
.ok_or(DecodeError::OutOfBoundsByte { i: 1 })?;
match ForkName::from_ssz_bytes(&[fork_byte])? {
ForkName::Base => Ok(SignedBeaconBlock::Base(
SignedBeaconBlockBase::from_ssz_bytes(body)?,
)),
ForkName::Altair => Ok(SignedBeaconBlock::Altair(
SignedBeaconBlockAltair::from_ssz_bytes(body)?,
)),
ForkName::Merge => Ok(SignedBeaconBlock::Merge(
SignedBeaconBlockMerge::from_ssz_bytes(body)?,
)),
ForkName::Capella => Ok(SignedBeaconBlock::Capella(
SignedBeaconBlockCapella::from_ssz_bytes(body)?,
)),
ForkName::Deneb => Ok(SignedBeaconBlock::Deneb(
SignedBeaconBlockDeneb::from_ssz_bytes(body)?,
)),
}
}
}
}
#[cfg(test)]
mod test {
use super::*;
@ -551,4 +650,38 @@ mod test {
assert_eq!(reconstructed, block);
}
}
#[test]
fn test_ssz_tagged_signed_beacon_block() {
type E = MainnetEthSpec;
let spec = &E::default_spec();
let sig = Signature::empty();
let blocks = vec![
SignedBeaconBlock::<E>::from_block(
BeaconBlock::Base(BeaconBlockBase::empty(spec)),
sig.clone(),
),
SignedBeaconBlock::from_block(
BeaconBlock::Altair(BeaconBlockAltair::empty(spec)),
sig.clone(),
),
SignedBeaconBlock::from_block(
BeaconBlock::Merge(BeaconBlockMerge::empty(spec)),
sig.clone(),
),
SignedBeaconBlock::from_block(
BeaconBlock::Capella(BeaconBlockCapella::empty(spec)),
sig.clone(),
),
SignedBeaconBlock::from_block(BeaconBlock::Deneb(BeaconBlockDeneb::empty(spec)), sig),
];
for block in blocks {
let encoded = ssz_tagged_signed_beacon_block::encode::as_ssz_bytes(&block);
let decoded = ssz_tagged_signed_beacon_block::decode::from_ssz_bytes::<E, _>(&encoded)
.expect("should decode");
assert_eq!(decoded, block);
}
}
}