v0.12 fork choice update (#1229)

* Incomplete scraps

* Add progress on new fork choice impl

* Further progress

* First complete compiling version

* Remove chain reference

* Add new lmd_ghost crate

* Start integrating into beacon chain

* Update `milagro_bls` to new release (#1183)

* Update milagro_bls to new release

Signed-off-by: Kirk Baird <baird.k@outlook.com>

* Tidy up fake cryptos

Signed-off-by: Kirk Baird <baird.k@outlook.com>

* move SecretHash to bls and put plaintext back

Signed-off-by: Kirk Baird <baird.k@outlook.com>

* Update state processing for v0.12

* Fix EF test runners for v0.12

* Fix some tests

* Fix broken attestation verification test

* More test fixes

* Rough beacon chain impl working

* Remove fork_choice_2

* Remove checkpoint manager

* Half finished ssz impl

* Add missed file

* Add persistence

* Tidy, fix some compile errors

* Remove RwLock from ProtoArrayForkChoice

* Fix store-based compile errors

* Add comments, tidy

* Move function out of ForkChoice struct

* Start testing

* More testing

* Fix compile error

* Tidy beacon_chain::fork_choice

* Queue attestations from the current slot

* Allow fork choice to handle prior-to-genesis start

* Improve error granularity

* Test attestation dequeuing

* Process attestations during block

* Store target root in fork choice

* Move fork choice verification into new crate

* Update tests

* Consensus updates for v0.12 (#1228)

* Update state processing for v0.12

* Fix EF test runners for v0.12

* Fix some tests

* Fix broken attestation verification test

* More test fixes

* Fix typo found in review

* Add `Block` struct to ProtoArray

* Start fixing get_ancestor

* Add rough progress on testing

* Get fork choice tests working

* Progress with testing

* Fix partialeq impl

* Move slot clock from fc_store

* Improve testing

* Add testing for best justified

* Add clone back to SystemTimeSlotClock

* Add balances test

* Start adding balances cache again

* Wire-in balances cache

* Improve tests

* Remove commented-out tests

* Remove beacon_chain::ForkChoice

* Rename crates

* Update wider codebase to new fork_choice layout

* Move advance_slot in test harness

* Tidy ForkChoice::update_time

* Fix verification tests

* Fix compile error with iter::once

* Fix fork choice tests

* Ensure block attestations are processed

* Fix failing beacon_chain tests

* Add first invalid block check

* Add finalized block check

* Progress with testing, new store builder

* Add fixes to get_ancestor

* Fix old genesis justification test

* Fix remaining fork choice tests

* Change root iteration method

* Move on_verified_block

* Remove unused method

* Start adding attestation verification tests

* Add invalid ffg target test

* Add target epoch test

* Add queued attestation test

* Remove old fork choice verification tests

* Tidy, add test

* Move fork choice lock drop

* Rename BeaconForkChoiceStore

* Add comments, tidy BeaconForkChoiceStore

* Update metrics, rename fork_choice_store.rs

* Remove genesis_block_root from ForkChoice

* Tidy

* Update fork_choice comments

* Tidy, add comments

* Tidy, simplify ForkChoice, fix compile issue

* Tidy, removed dead file

* Increase http request timeout

* Fix failing rest_api test

* Set HTTP timeout back to 5s

* Apply fix to get_ancestor

* Address Michael's comments

* Fix typo

* Revert "Fix broken attestation verification test"

This reverts commit 722cdc903b12611de27916a57eeecfa3224f2279.

Co-authored-by: Kirk Baird <baird.k@outlook.com>
Co-authored-by: Michael Sproul <michael@sigmaprime.io>
This commit is contained in:
Paul Hauner 2020-06-17 11:10:22 +10:00 committed by GitHub
parent 1a4de898bc
commit 764cb2d32a
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
51 changed files with 2641 additions and 1376 deletions

24
Cargo.lock generated
View File

@ -252,6 +252,7 @@ dependencies = [
"eth2_ssz",
"eth2_ssz_derive",
"eth2_ssz_types",
"fork_choice",
"futures 0.3.5",
"genesis",
"integer-sqrt",
@ -262,7 +263,7 @@ dependencies = [
"merkle_proof",
"operation_pool",
"parking_lot 0.10.2",
"proto_array_fork_choice",
"proto_array",
"rand 0.7.3",
"rayon",
"safe_arith",
@ -1531,6 +1532,21 @@ version = "0.1.1"
source = "registry+https://github.com/rust-lang/crates.io-index"
checksum = "00b0228411908ca8685dba7fc2cdd70ec9990a6e753e89b6ac91a84c40fbaf4b"
[[package]]
name = "fork_choice"
version = "0.1.0"
dependencies = [
"beacon_chain",
"eth2_ssz",
"eth2_ssz_derive",
"proto_array",
"slot_clock",
"state_processing",
"store",
"tree_hash",
"types",
]
[[package]]
name = "fuchsia-cprng"
version = "0.1.1"
@ -3413,13 +3429,12 @@ dependencies = [
]
[[package]]
name = "proto_array_fork_choice"
name = "proto_array"
version = "0.2.0"
dependencies = [
"eth2_ssz",
"eth2_ssz_derive",
"itertools 0.9.0",
"parking_lot 0.10.2",
"serde",
"serde_derive",
"serde_yaml",
@ -3714,7 +3729,7 @@ dependencies = [
"futures 0.3.5",
"hex 0.4.2",
"operation_pool",
"proto_array_fork_choice",
"proto_array",
"reqwest",
"rest_types",
"serde",
@ -4584,6 +4599,7 @@ dependencies = [
"db-key",
"eth2_ssz",
"eth2_ssz_derive",
"fork_choice",
"itertools 0.9.0",
"lazy_static",
"leveldb",

View File

@ -32,7 +32,8 @@ members = [
"consensus/cached_tree_hash",
"consensus/int_to_bytes",
"consensus/proto_array_fork_choice",
"consensus/fork_choice",
"consensus/proto_array",
"consensus/safe_arith",
"consensus/ssz",
"consensus/ssz_derive",

View File

@ -40,15 +40,13 @@ futures = "0.3.5"
genesis = { path = "../genesis" }
integer-sqrt = "0.1.3"
rand = "0.7.3"
proto_array_fork_choice = { path = "../../consensus/proto_array_fork_choice" }
proto_array = { path = "../../consensus/proto_array" }
lru = "0.5.1"
tempfile = "3.1.0"
bitvec = "0.17.4"
bls = { path = "../../crypto/bls" }
safe_arith = { path = "../../consensus/safe_arith" }
fork_choice = { path = "../../consensus/fork_choice" }
environment = { path = "../../lighthouse/environment" }
bus = "2.2.3"
[dev-dependencies]
lazy_static = "1.4.0"

View File

@ -23,7 +23,7 @@
//! -------------------------------------
//! |
//! ▼
//! ForkChoiceVerifiedAttestation
//! impl SignatureVerifiedAttestation
//! ```
use crate::{
@ -158,65 +158,21 @@ impl<T: BeaconChainTypes> Clone for VerifiedUnaggregatedAttestation<T> {
}
}
/// Wraps an `indexed_attestation` that is valid for application to fork choice. The
/// `indexed_attestation` will have been generated via the `VerifiedAggregatedAttestation` or
/// `VerifiedUnaggregatedAttestation` wrappers.
pub struct ForkChoiceVerifiedAttestation<'a, T: BeaconChainTypes> {
indexed_attestation: &'a IndexedAttestation<T::EthSpec>,
}
/// A helper trait implemented on wrapper types that can be progressed to a state where they can be
/// verified for application to fork choice.
pub trait IntoForkChoiceVerifiedAttestation<'a, T: BeaconChainTypes> {
fn into_fork_choice_verified_attestation(
&'a self,
chain: &BeaconChain<T>,
) -> Result<ForkChoiceVerifiedAttestation<'a, T>, Error>;
pub trait SignatureVerifiedAttestation<T: BeaconChainTypes> {
fn indexed_attestation(&self) -> &IndexedAttestation<T::EthSpec>;
}
impl<'a, T: BeaconChainTypes> IntoForkChoiceVerifiedAttestation<'a, T>
for VerifiedAggregatedAttestation<T>
{
/// Progresses the `VerifiedAggregatedAttestation` to a stage where it is valid for application
/// to the fork-choice rule (or not).
fn into_fork_choice_verified_attestation(
&'a self,
chain: &BeaconChain<T>,
) -> Result<ForkChoiceVerifiedAttestation<T>, Error> {
ForkChoiceVerifiedAttestation::from_signature_verified_components(
&self.indexed_attestation,
chain,
)
impl<'a, T: BeaconChainTypes> SignatureVerifiedAttestation<T> for VerifiedAggregatedAttestation<T> {
fn indexed_attestation(&self) -> &IndexedAttestation<T::EthSpec> {
&self.indexed_attestation
}
}
impl<'a, T: BeaconChainTypes> IntoForkChoiceVerifiedAttestation<'a, T>
for VerifiedUnaggregatedAttestation<T>
{
/// Progresses the `Attestation` to a stage where it is valid for application to the
/// fork-choice rule (or not).
fn into_fork_choice_verified_attestation(
&'a self,
chain: &BeaconChain<T>,
) -> Result<ForkChoiceVerifiedAttestation<T>, Error> {
ForkChoiceVerifiedAttestation::from_signature_verified_components(
&self.indexed_attestation,
chain,
)
}
}
impl<'a, T: BeaconChainTypes> IntoForkChoiceVerifiedAttestation<'a, T>
for ForkChoiceVerifiedAttestation<'a, T>
{
/// Simply returns itself.
fn into_fork_choice_verified_attestation(
&'a self,
_: &BeaconChain<T>,
) -> Result<ForkChoiceVerifiedAttestation<T>, Error> {
Ok(Self {
indexed_attestation: self.indexed_attestation,
})
impl<T: BeaconChainTypes> SignatureVerifiedAttestation<T> for VerifiedUnaggregatedAttestation<T> {
fn indexed_attestation(&self) -> &IndexedAttestation<T::EthSpec> {
&self.indexed_attestation
}
}
@ -344,14 +300,6 @@ impl<T: BeaconChainTypes> VerifiedAggregatedAttestation<T> {
chain.add_to_block_inclusion_pool(self)
}
/// A helper function to add this aggregate to `beacon_chain.fork_choice`.
pub fn add_to_fork_choice(
&self,
chain: &BeaconChain<T>,
) -> Result<ForkChoiceVerifiedAttestation<T>, Error> {
chain.apply_attestation_to_fork_choice(self)
}
/// Returns the underlying `attestation` for the `signed_aggregate`.
pub fn attestation(&self) -> &Attestation<T::EthSpec> {
&self.signed_aggregate.message.aggregate
@ -449,114 +397,6 @@ impl<T: BeaconChainTypes> VerifiedUnaggregatedAttestation<T> {
}
}
impl<'a, T: BeaconChainTypes> ForkChoiceVerifiedAttestation<'a, T> {
/// Returns `Ok(Self)` if the `attestation` is valid to be applied to the beacon chain fork
/// choice.
///
/// The supplied `indexed_attestation` MUST have a valid signature, this function WILL NOT
/// CHECK THE SIGNATURE. Use the `VerifiedAggregatedAttestation` or
/// `VerifiedUnaggregatedAttestation` structs to do signature verification.
fn from_signature_verified_components(
indexed_attestation: &'a IndexedAttestation<T::EthSpec>,
chain: &BeaconChain<T>,
) -> Result<Self, Error> {
// There is no point in processing an attestation with an empty bitfield. Reject
// it immediately.
//
// This is not in the specification, however it should be transparent to other nodes. We
// return early here to avoid wasting precious resources verifying the rest of it.
if indexed_attestation.attesting_indices.len() == 0 {
return Err(Error::EmptyAggregationBitfield);
}
let slot_now = chain.slot()?;
let epoch_now = slot_now.epoch(T::EthSpec::slots_per_epoch());
let target = indexed_attestation.data.target.clone();
// Attestation must be from the current or previous epoch.
if target.epoch > epoch_now {
return Err(Error::FutureEpoch {
attestation_epoch: target.epoch,
current_epoch: epoch_now,
});
} else if target.epoch + 1 < epoch_now {
return Err(Error::PastEpoch {
attestation_epoch: target.epoch,
current_epoch: epoch_now,
});
}
if target.epoch
!= indexed_attestation
.data
.slot
.epoch(T::EthSpec::slots_per_epoch())
{
return Err(Error::BadTargetEpoch);
}
// Attestation target must be for a known block.
if !chain.fork_choice.contains_block(&target.root) {
return Err(Error::UnknownTargetRoot(target.root));
}
// TODO: we're not testing an assert from the spec:
//
// `assert get_current_slot(store) >= compute_start_slot_at_epoch(target.epoch)`
//
// I think this check is redundant and I've raised an issue here:
//
// https://github.com/ethereum/eth2.0-specs/pull/1755
//
// To resolve this todo, observe the outcome of the above PR.
// Load the slot and state root for `attestation.data.beacon_block_root`.
//
// This indirectly checks to see if the `attestation.data.beacon_block_root` is in our fork
// choice. Any known, non-finalized block should be in fork choice, so this check
// immediately filters out attestations that attest to a block that has not been processed.
//
// Attestations must be for a known block. If the block is unknown, we simply drop the
// attestation and do not delay consideration for later.
let (block_slot, _state_root) = chain
.fork_choice
.block_slot_and_state_root(&indexed_attestation.data.beacon_block_root)
.ok_or_else(|| Error::UnknownHeadBlock {
beacon_block_root: indexed_attestation.data.beacon_block_root,
})?;
// TODO: currently we do not check the FFG source/target. This is what the spec dictates
// but it seems wrong.
//
// I have opened an issue on the specs repo for this:
//
// https://github.com/ethereum/eth2.0-specs/issues/1636
//
// We should revisit this code once that issue has been resolved.
// Attestations must not be for blocks in the future. If this is the case, the attestation
// should not be considered.
if block_slot > indexed_attestation.data.slot {
return Err(Error::AttestsToFutureBlock {
block: block_slot,
attestation: indexed_attestation.data.slot,
});
}
// Note: we're not checking the "attestations can only affect the fork choice of subsequent
// slots" part of the spec, we do this upstream.
Ok(Self {
indexed_attestation,
})
}
/// Returns the wrapped `IndexedAttestation`.
pub fn indexed_attestation(&self) -> &IndexedAttestation<T::EthSpec> {
&self.indexed_attestation
}
}
/// Returns `Ok(())` if the `attestation.data.beacon_block_root` is known to this chain.
///
/// The block root may not be known for two reasons:
@ -573,6 +413,7 @@ fn verify_head_block_is_known<T: BeaconChainTypes>(
) -> Result<(), Error> {
if chain
.fork_choice
.read()
.contains_block(&attestation.data.beacon_block_root)
{
Ok(())
@ -765,9 +606,10 @@ where
// processing an attestation that does not include our latest finalized block in its chain.
//
// We do not delay consideration for later, we simply drop the attestation.
let (target_block_slot, target_block_state_root) = chain
let target_block = chain
.fork_choice
.block_slot_and_state_root(&target.root)
.read()
.get_block(&target.root)
.ok_or_else(|| Error::UnknownTargetRoot(target.root))?;
// Obtain the shuffling cache, timing how long we wait.
@ -800,15 +642,15 @@ where
chain.log,
"Attestation processing cache miss";
"attn_epoch" => attestation_epoch.as_u64(),
"target_block_epoch" => target_block_slot.epoch(T::EthSpec::slots_per_epoch()).as_u64(),
"target_block_epoch" => target_block.slot.epoch(T::EthSpec::slots_per_epoch()).as_u64(),
);
let state_read_timer =
metrics::start_timer(&metrics::ATTESTATION_PROCESSING_STATE_READ_TIMES);
let mut state = chain
.get_state(&target_block_state_root, Some(target_block_slot))?
.ok_or_else(|| BeaconChainError::MissingBeaconState(target_block_state_root))?;
.get_state(&target_block.state_root, Some(target_block.slot))?
.ok_or_else(|| BeaconChainError::MissingBeaconState(target_block.state_root))?;
metrics::stop_timer(state_read_timer);
let state_skip_timer =

View File

@ -1,6 +1,6 @@
use crate::attestation_verification::{
Error as AttestationError, ForkChoiceVerifiedAttestation, IntoForkChoiceVerifiedAttestation,
VerifiedAggregatedAttestation, VerifiedUnaggregatedAttestation,
Error as AttestationError, SignatureVerifiedAttestation, VerifiedAggregatedAttestation,
VerifiedUnaggregatedAttestation,
};
use crate::block_verification::{
check_block_relevancy, get_block_root, signature_verify_chain_segment, BlockError,
@ -9,7 +9,6 @@ use crate::block_verification::{
use crate::errors::{BeaconChainError as Error, BlockProductionError};
use crate::eth1_chain::{Eth1Chain, Eth1ChainBackend};
use crate::events::{EventHandler, EventKind};
use crate::fork_choice::{Error as ForkChoiceError, ForkChoice};
use crate::head_tracker::HeadTracker;
use crate::metrics;
use crate::migrate::Migrate;
@ -18,17 +17,24 @@ use crate::observed_attestations::{Error as AttestationObservationError, Observe
use crate::observed_attesters::{ObservedAggregators, ObservedAttesters};
use crate::observed_block_producers::ObservedBlockProducers;
use crate::persisted_beacon_chain::PersistedBeaconChain;
use crate::persisted_fork_choice::PersistedForkChoice;
use crate::shuffling_cache::ShufflingCache;
use crate::snapshot_cache::SnapshotCache;
use crate::timeout_rw_lock::TimeoutRwLock;
use crate::validator_pubkey_cache::ValidatorPubkeyCache;
use crate::BeaconForkChoiceStore;
use crate::BeaconSnapshot;
use fork_choice::ForkChoice;
use operation_pool::{OperationPool, PersistedOperationPool};
use parking_lot::RwLock;
use slog::{crit, debug, error, info, trace, warn, Logger};
use slot_clock::SlotClock;
use state_processing::per_block_processing::errors::{
AttestationValidationError, AttesterSlashingValidationError, ExitValidationError,
ProposerSlashingValidationError,
use state_processing::{
common::get_indexed_attestation,
per_block_processing::errors::{
AttestationValidationError, AttesterSlashingValidationError, ExitValidationError,
ProposerSlashingValidationError,
},
};
use state_processing::{per_block_processing, per_slot_processing, BlockSignatureStrategy};
use std::borrow::Cow;
@ -42,6 +48,8 @@ use store::iter::{BlockRootsIterator, ParentRootBlockIterator, StateRootsIterato
use store::{Error as DBError, Store};
use types::*;
pub type ForkChoiceError = fork_choice::Error<crate::ForkChoiceStoreError>;
// Text included in blocks.
// Must be 32-bytes or panic.
//
@ -193,7 +201,7 @@ pub struct BeaconChain<T: BeaconChainTypes> {
pub genesis_validators_root: Hash256,
/// A state-machine that is updated with information from the network and chooses a canonical
/// head block.
pub fork_choice: ForkChoice<T>,
pub fork_choice: RwLock<ForkChoice<BeaconForkChoiceStore<T::Store, T::EthSpec>, T::EthSpec>>,
/// A handler for events generated by the beacon chain.
pub event_handler: T::EventHandler,
/// Used to track the heads of the beacon chain.
@ -238,11 +246,18 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
let fork_choice_timer = metrics::start_timer(&metrics::PERSIST_FORK_CHOICE);
let fork_choice = self.fork_choice.read();
self.store.put_item(
&Hash256::from_slice(&FORK_CHOICE_DB_KEY),
&self.fork_choice.as_ssz_container(),
&PersistedForkChoice {
fork_choice: fork_choice.to_persisted(),
fork_choice_store: fork_choice.fc_store().to_persisted(),
},
)?;
drop(fork_choice);
metrics::stop_timer(fork_choice_timer);
let head_timer = metrics::start_timer(&metrics::PERSIST_HEAD);
@ -261,21 +276,19 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
/// This operation is typically slow and causes a lot of allocations. It should be used
/// sparingly.
pub fn persist_op_pool(&self) -> Result<(), Error> {
let timer = metrics::start_timer(&metrics::PERSIST_OP_POOL);
let _timer = metrics::start_timer(&metrics::PERSIST_OP_POOL);
self.store.put_item(
&Hash256::from_slice(&OP_POOL_DB_KEY),
&PersistedOperationPool::from_operation_pool(&self.op_pool),
)?;
metrics::stop_timer(timer);
Ok(())
}
/// Persists `self.eth1_chain` and its caches to disk.
pub fn persist_eth1_cache(&self) -> Result<(), Error> {
let timer = metrics::start_timer(&metrics::PERSIST_OP_POOL);
let _timer = metrics::start_timer(&metrics::PERSIST_OP_POOL);
if let Some(eth1_chain) = self.eth1_chain.as_ref() {
self.store.put_item(
@ -284,8 +297,6 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
)?;
}
metrics::stop_timer(timer);
Ok(())
}
@ -876,23 +887,20 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
/// Accepts some attestation-type object and attempts to verify it in the context of fork
/// choice. If it is valid it is applied to `self.fork_choice`.
///
/// Common items that implement `IntoForkChoiceVerifiedAttestation`:
/// Common items that implement `SignatureVerifiedAttestation`:
///
/// - `VerifiedUnaggregatedAttestation`
/// - `VerifiedAggregatedAttestation`
/// - `ForkChoiceVerifiedAttestation`
pub fn apply_attestation_to_fork_choice<'a>(
&self,
unverified_attestation: &'a impl IntoForkChoiceVerifiedAttestation<'a, T>,
) -> Result<ForkChoiceVerifiedAttestation<'a, T>, AttestationError> {
let _timer = metrics::start_timer(&metrics::ATTESTATION_PROCESSING_APPLY_TO_FORK_CHOICE);
verified: &'a impl SignatureVerifiedAttestation<T>,
) -> Result<(), Error> {
let _timer = metrics::start_timer(&metrics::FORK_CHOICE_PROCESS_ATTESTATION_TIMES);
let verified = unverified_attestation.into_fork_choice_verified_attestation(self)?;
let indexed_attestation = verified.indexed_attestation();
self.fork_choice
.process_indexed_attestation(indexed_attestation)
.map_err(|e| Error::from(e))?;
Ok(verified)
.write()
.on_attestation(self.slot()?, verified.indexed_attestation())
.map_err(Into::into)
}
/// Accepts an `VerifiedUnaggregatedAttestation` and attempts to apply it to the "naive
@ -1028,8 +1036,10 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
// pivot block is the same as the current state's pivot block. If it is, then the
// attestation's shuffling is the same as the current state's.
// To account for skipped slots, find the first block at *or before* the pivot slot.
let fork_choice_lock = self.fork_choice.core_proto_array();
let fork_choice_lock = self.fork_choice.read();
let pivot_block_root = fork_choice_lock
.proto_array()
.core_proto_array()
.iter_block_roots(block_root)
.find(|(_, slot)| *slot <= pivot_slot)
.map(|(block_root, _)| block_root);
@ -1325,7 +1335,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
unverified_block: B,
) -> Result<Hash256, BlockError> {
// Start the Prometheus timer.
let full_timer = metrics::start_timer(&metrics::BLOCK_PROCESSING_TIMES);
let _full_timer = metrics::start_timer(&metrics::BLOCK_PROCESSING_TIMES);
// Increment the Prometheus counter for block processing requests.
metrics::inc_counter(&metrics::BLOCK_PROCESSING_REQUESTS);
@ -1393,9 +1403,6 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
}
};
// Stop the Prometheus timer.
metrics::stop_timer(full_timer);
result
}
@ -1414,6 +1421,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
let state = fully_verified_block.state;
let parent_block = fully_verified_block.parent_block;
let intermediate_states = fully_verified_block.intermediate_states;
let current_slot = self.slot()?;
let attestation_observation_timer =
metrics::start_timer(&metrics::BLOCK_PROCESSING_ATTESTATION_OBSERVATION);
@ -1433,9 +1441,6 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
metrics::stop_timer(attestation_observation_timer);
let fork_choice_register_timer =
metrics::start_timer(&metrics::BLOCK_PROCESSING_FORK_CHOICE_REGISTER);
// If there are new validators in this block, update our pubkey cache.
//
// We perform this _before_ adding the block to fork choice because the pubkey cache is
@ -1471,20 +1476,35 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
shuffling_cache.insert(state.current_epoch(), target_root, committee_cache);
}
let mut fork_choice = self.fork_choice.write();
// Register the new block with the fork choice service.
if let Err(e) = self
.fork_choice
.process_block(self, &state, block, block_root)
{
error!(
self.log,
"Add block to fork choice failed";
"block_root" => format!("{}", block_root),
"error" => format!("{:?}", e),
)
let _fork_choice_block_timer =
metrics::start_timer(&metrics::FORK_CHOICE_PROCESS_BLOCK_TIMES);
fork_choice
.on_block(current_slot, block, block_root, &state)
.map_err(|e| BlockError::BeaconChainError(e.into()))?;
}
metrics::stop_timer(fork_choice_register_timer);
// Register each attestation in the block with the fork choice service.
for attestation in &block.body.attestations[..] {
let _fork_choice_attestation_timer =
metrics::start_timer(&metrics::FORK_CHOICE_PROCESS_ATTESTATION_TIMES);
let committee =
state.get_beacon_committee(attestation.data.slot, attestation.data.index)?;
let indexed_attestation = get_indexed_attestation(committee.committee, attestation)
.map_err(|e| BlockError::BeaconChainError(e.into()))?;
match fork_choice.on_attestation(current_slot, &indexed_attestation) {
Ok(()) => Ok(()),
// Ignore invalid attestations whilst importing attestations from a block. The
// block might be very old and therefore the attestations useless to fork choice.
Err(ForkChoiceError::InvalidAttestation(_)) => Ok(()),
Err(e) => Err(BlockError::BeaconChainError(e.into())),
}?;
}
metrics::observe(
&metrics::OPERATIONS_PER_BLOCK_ATTESTATION,
@ -1506,6 +1526,10 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
self.store.put_state(&block.state_root, &state)?;
self.store.put_block(&block_root, signed_block.clone())?;
// The fork choice write-lock is dropped *after* the on-disk database has been updated.
// This prevents inconsistency between the two at the expense of concurrency.
drop(fork_choice);
let parent_root = block.parent_root;
let slot = block.slot;
@ -1674,7 +1698,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
/// Execute the fork choice algorithm and enthrone the result as the canonical head.
pub fn fork_choice(&self) -> Result<(), Error> {
metrics::inc_counter(&metrics::FORK_CHOICE_REQUESTS);
let overall_timer = metrics::start_timer(&metrics::FORK_CHOICE_TIMES);
let _timer = metrics::start_timer(&metrics::FORK_CHOICE_TIMES);
let result = self.fork_choice_internal();
@ -1682,14 +1706,12 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
metrics::inc_counter(&metrics::FORK_CHOICE_ERRORS);
}
metrics::stop_timer(overall_timer);
result
}
fn fork_choice_internal(&self) -> Result<(), Error> {
// Determine the root of the block that is the head of the chain.
let beacon_block_root = self.fork_choice.find_head(&self)?;
let beacon_block_root = self.fork_choice.write().get_head(self.slot()?)?;
let current_head = self.head_info()?;
let old_finalized_root = current_head.finalized_checkpoint.root;
@ -1869,7 +1891,7 @@ impl<T: BeaconChainTypes> BeaconChain<T> {
new_epoch: new_finalized_epoch,
})
} else {
self.fork_choice.prune()?;
self.fork_choice.write().prune()?;
self.observed_block_producers
.prune(new_finalized_epoch.start_slot(T::EthSpec::slots_per_epoch()));

View File

@ -0,0 +1,334 @@
//! Defines the `BeaconForkChoiceStore` which provides the persistent storage for the `ForkChoice`
//! struct.
//!
//! Additionally, the private `BalancesCache` struct is defined; a cache designed to avoid database
//! reads when fork choice requires the validator balances of the justified state.
use crate::{metrics, BeaconSnapshot};
use fork_choice::ForkChoiceStore;
use ssz_derive::{Decode, Encode};
use std::marker::PhantomData;
use std::sync::Arc;
use store::{Error as StoreError, Store};
use types::{
BeaconBlock, BeaconState, BeaconStateError, Checkpoint, EthSpec, Hash256, SignedBeaconBlock,
Slot,
};
#[derive(Debug)]
pub enum Error {
UnableToReadSlot,
UnableToReadTime,
InvalidGenesisSnapshot(Slot),
AncestorUnknown { ancestor_slot: Slot },
UninitializedBestJustifiedBalances,
FailedToReadBlock(StoreError),
MissingBlock(Hash256),
FailedToReadState(StoreError),
MissingState(Hash256),
InvalidPersistedBytes(ssz::DecodeError),
BeaconStateError(BeaconStateError),
}
impl From<BeaconStateError> for Error {
fn from(e: BeaconStateError) -> Self {
Error::BeaconStateError(e)
}
}
/// The number of validator balance sets that are cached within `BalancesCache`.
const MAX_BALANCE_CACHE_SIZE: usize = 4;
/// Returns the effective balances for every validator in the given `state`.
///
/// Any validator who is not active in the epoch of the given `state` is assigned a balance of
/// zero.
pub fn get_effective_balances<T: EthSpec>(state: &BeaconState<T>) -> Vec<u64> {
state
.validators
.iter()
.map(|validator| {
if validator.is_active_at(state.current_epoch()) {
validator.effective_balance
} else {
0
}
})
.collect()
}
/// An item that is stored in the `BalancesCache`.
#[derive(PartialEq, Clone, Debug, Encode, Decode)]
struct CacheItem {
/// The block root at which `self.balances` are valid.
block_root: Hash256,
/// The effective balances from a `BeaconState` validator registry.
balances: Vec<u64>,
}
/// Provides a cache to avoid reading `BeaconState` from disk when updating the current justified
/// checkpoint.
///
/// It is effectively a mapping of `epoch_boundary_block_root -> state.balances`.
#[derive(PartialEq, Clone, Default, Debug, Encode, Decode)]
struct BalancesCache {
items: Vec<CacheItem>,
}
impl BalancesCache {
/// Inspect the given `state` and determine the root of the block at the first slot of
/// `state.current_epoch`. If there is not already some entry for the given block root, then
/// add the effective balances from the `state` to the cache.
pub fn process_state<E: EthSpec>(
&mut self,
block_root: Hash256,
state: &BeaconState<E>,
) -> Result<(), Error> {
// We are only interested in balances from states that are at the start of an epoch,
// because this is where the `current_justified_checkpoint.root` will point.
if !Self::is_first_block_in_epoch(block_root, state)? {
return Ok(());
}
let epoch_boundary_slot = state.current_epoch().start_slot(E::slots_per_epoch());
let epoch_boundary_root = if epoch_boundary_slot == state.slot {
block_root
} else {
// This call remains sensible as long as `state.block_roots` is larger than a single
// epoch.
*state.get_block_root(epoch_boundary_slot)?
};
if self.position(epoch_boundary_root).is_none() {
let item = CacheItem {
block_root: epoch_boundary_root,
balances: get_effective_balances(state),
};
if self.items.len() == MAX_BALANCE_CACHE_SIZE {
self.items.remove(0);
}
self.items.push(item);
}
Ok(())
}
/// Returns `true` if the given `block_root` is the first/only block to have been processed in
/// the epoch of the given `state`.
///
/// We can determine if it is the first block by looking back through `state.block_roots` to
/// see if there is a block in the current epoch with a different root.
fn is_first_block_in_epoch<E: EthSpec>(
block_root: Hash256,
state: &BeaconState<E>,
) -> Result<bool, Error> {
let mut prior_block_found = false;
for slot in state.current_epoch().slot_iter(E::slots_per_epoch()) {
if slot < state.slot {
if *state.get_block_root(slot)? != block_root {
prior_block_found = true;
break;
}
} else {
break;
}
}
Ok(!prior_block_found)
}
fn position(&self, block_root: Hash256) -> Option<usize> {
self.items
.iter()
.position(|item| item.block_root == block_root)
}
/// Get the balances for the given `block_root`, if any.
///
/// If some balances are found, they are removed from the cache.
pub fn get(&mut self, block_root: Hash256) -> Option<Vec<u64>> {
let i = self.position(block_root)?;
Some(self.items.remove(i).balances)
}
}
/// Implements `fork_choice::ForkChoiceStore` in order to provide a persistent backing to the
/// `fork_choice::ForkChoice` struct.
#[derive(Debug)]
pub struct BeaconForkChoiceStore<S, E> {
store: Arc<S>,
balances_cache: BalancesCache,
time: Slot,
finalized_checkpoint: Checkpoint,
justified_checkpoint: Checkpoint,
justified_balances: Vec<u64>,
best_justified_checkpoint: Checkpoint,
_phantom: PhantomData<E>,
}
impl<S, E> PartialEq for BeaconForkChoiceStore<S, E> {
/// This implementation ignores the `store` and `slot_clock`.
fn eq(&self, other: &Self) -> bool {
self.balances_cache == other.balances_cache
&& self.time == other.time
&& self.finalized_checkpoint == other.finalized_checkpoint
&& self.justified_checkpoint == other.justified_checkpoint
&& self.justified_balances == other.justified_balances
&& self.best_justified_checkpoint == other.best_justified_checkpoint
}
}
impl<S: Store<E>, E: EthSpec> BeaconForkChoiceStore<S, E> {
/// Initialize `Self` from some `anchor` checkpoint which may or may not be the genesis state.
///
/// ## Specification
///
/// Equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.1/specs/phase0/fork-choice.md#get_forkchoice_store
///
/// ## Notes:
///
/// It is assumed that `anchor` is already persisted in `store`.
pub fn get_forkchoice_store(store: Arc<S>, anchor: &BeaconSnapshot<E>) -> Self {
let anchor_state = &anchor.beacon_state;
let mut anchor_block_header = anchor_state.latest_block_header.clone();
if anchor_block_header.state_root == Hash256::zero() {
anchor_block_header.state_root = anchor.beacon_state_root;
}
let anchor_root = anchor_block_header.canonical_root();
let anchor_epoch = anchor_state.current_epoch();
let justified_checkpoint = Checkpoint {
epoch: anchor_epoch,
root: anchor_root,
};
let finalized_checkpoint = justified_checkpoint;
Self {
store,
balances_cache: <_>::default(),
time: anchor_state.slot,
justified_checkpoint,
justified_balances: anchor_state.balances.clone().into(),
finalized_checkpoint,
best_justified_checkpoint: justified_checkpoint,
_phantom: PhantomData,
}
}
/// Save the current state of `Self` to a `PersistedForkChoiceStore` which can be stored to the
/// on-disk database.
pub fn to_persisted(&self) -> PersistedForkChoiceStore {
PersistedForkChoiceStore {
balances_cache: self.balances_cache.clone(),
time: self.time,
finalized_checkpoint: self.finalized_checkpoint,
justified_checkpoint: self.justified_checkpoint,
justified_balances: self.justified_balances.clone(),
best_justified_checkpoint: self.best_justified_checkpoint,
}
}
/// Restore `Self` from a previously-generated `PersistedForkChoiceStore`.
pub fn from_persisted(
persisted: PersistedForkChoiceStore,
store: Arc<S>,
) -> Result<Self, Error> {
Ok(Self {
store,
balances_cache: persisted.balances_cache,
time: persisted.time,
finalized_checkpoint: persisted.finalized_checkpoint,
justified_checkpoint: persisted.justified_checkpoint,
justified_balances: persisted.justified_balances,
best_justified_checkpoint: persisted.best_justified_checkpoint,
_phantom: PhantomData,
})
}
}
impl<S: Store<E>, E: EthSpec> ForkChoiceStore<E> for BeaconForkChoiceStore<S, E> {
type Error = Error;
fn get_current_slot(&self) -> Slot {
self.time
}
fn set_current_slot(&mut self, slot: Slot) {
self.time = slot
}
fn on_verified_block(
&mut self,
_block: &BeaconBlock<E>,
block_root: Hash256,
state: &BeaconState<E>,
) -> Result<(), Self::Error> {
self.balances_cache.process_state(block_root, state)
}
fn justified_checkpoint(&self) -> &Checkpoint {
&self.justified_checkpoint
}
fn justified_balances(&self) -> &[u64] {
&self.justified_balances
}
fn best_justified_checkpoint(&self) -> &Checkpoint {
&self.best_justified_checkpoint
}
fn finalized_checkpoint(&self) -> &Checkpoint {
&self.finalized_checkpoint
}
fn set_finalized_checkpoint(&mut self, checkpoint: Checkpoint) {
self.finalized_checkpoint = checkpoint
}
fn set_justified_checkpoint(&mut self, checkpoint: Checkpoint) -> Result<(), Error> {
self.justified_checkpoint = checkpoint;
if let Some(balances) = self.balances_cache.get(self.justified_checkpoint.root) {
metrics::inc_counter(&metrics::BALANCES_CACHE_HITS);
self.justified_balances = balances;
} else {
metrics::inc_counter(&metrics::BALANCES_CACHE_MISSES);
let justified_block = self
.store
.get_item::<SignedBeaconBlock<E>>(&self.justified_checkpoint.root)
.map_err(Error::FailedToReadBlock)?
.ok_or_else(|| Error::MissingBlock(self.justified_checkpoint.root))?
.message;
self.justified_balances = self
.store
.get_state(&justified_block.state_root, Some(justified_block.slot))
.map_err(Error::FailedToReadState)?
.ok_or_else(|| Error::MissingState(justified_block.state_root))?
.balances
.into();
}
Ok(())
}
fn set_best_justified_checkpoint(&mut self, checkpoint: Checkpoint) {
self.best_justified_checkpoint = checkpoint
}
}
/// A container which allows persisting the `BeaconForkChoiceStore` to the on-disk database.
#[derive(Encode, Decode)]
pub struct PersistedForkChoiceStore {
balances_cache: BalancesCache,
time: Slot,
finalized_checkpoint: Checkpoint,
justified_checkpoint: Checkpoint,
justified_balances: Vec<u64>,
best_justified_checkpoint: Checkpoint,
}

View File

@ -530,7 +530,11 @@ impl<T: BeaconChainTypes> FullyVerifiedBlock<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).
if !chain.fork_choice.contains_block(&block.parent_root()) {
if !chain
.fork_choice
.read()
.contains_block(&block.parent_root())
{
return Err(BlockError::ParentUnknown(block.parent_root()));
}
@ -727,7 +731,7 @@ pub fn check_block_relevancy<T: BeaconChainTypes>(
// Check if the block is already known. We know it is post-finalization, so it is
// sufficient to check the fork choice.
if chain.fork_choice.contains_block(&block_root) {
if chain.fork_choice.read().contains_block(&block_root) {
return Err(BlockError::BlockIsAlreadyKnown);
}
@ -767,7 +771,7 @@ fn load_parent<T: BeaconChainTypes>(
// 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).
if !chain.fork_choice.contains_block(&block.parent_root) {
if !chain.fork_choice.read().contains_block(&block.parent_root) {
return Err(BlockError::ParentUnknown(block.parent_root));
}

View File

@ -3,21 +3,22 @@ use crate::beacon_chain::{
};
use crate::eth1_chain::{CachingEth1Backend, SszEth1};
use crate::events::NullEventHandler;
use crate::fork_choice::SszForkChoice;
use crate::head_tracker::HeadTracker;
use crate::migrate::Migrate;
use crate::persisted_beacon_chain::PersistedBeaconChain;
use crate::persisted_fork_choice::PersistedForkChoice;
use crate::shuffling_cache::ShufflingCache;
use crate::snapshot_cache::{SnapshotCache, DEFAULT_SNAPSHOT_CACHE_SIZE};
use crate::timeout_rw_lock::TimeoutRwLock;
use crate::validator_pubkey_cache::ValidatorPubkeyCache;
use crate::{
BeaconChain, BeaconChainTypes, BeaconSnapshot, Eth1Chain, Eth1ChainBackend, EventHandler,
ForkChoice,
BeaconChain, BeaconChainTypes, BeaconForkChoiceStore, BeaconSnapshot, Eth1Chain,
Eth1ChainBackend, EventHandler,
};
use eth1::Config as Eth1Config;
use fork_choice::ForkChoice;
use operation_pool::{OperationPool, PersistedOperationPool};
use proto_array_fork_choice::ProtoArrayForkChoice;
use parking_lot::RwLock;
use slog::{info, Logger};
use slot_clock::{SlotClock, TestingSlotClock};
use std::marker::PhantomData;
@ -79,7 +80,6 @@ pub struct BeaconChainBuilder<T: BeaconChainTypes> {
pub finalized_snapshot: Option<BeaconSnapshot<T::EthSpec>>,
genesis_block_root: Option<Hash256>,
op_pool: Option<OperationPool<T::EthSpec>>,
fork_choice: Option<ForkChoice<T>>,
eth1_chain: Option<Eth1Chain<T::Eth1Chain, T::EthSpec, T::Store>>,
event_handler: Option<T::EventHandler>,
slot_clock: Option<T::SlotClock>,
@ -116,7 +116,6 @@ where
finalized_snapshot: None,
genesis_block_root: None,
op_pool: None,
fork_choice: None,
eth1_chain: None,
event_handler: None,
slot_clock: None,
@ -386,6 +385,13 @@ where
let log = self
.log
.ok_or_else(|| "Cannot build without a logger".to_string())?;
let slot_clock = self
.slot_clock
.ok_or_else(|| "Cannot build without a slot_clock.".to_string())?;
let store = self
.store
.clone()
.ok_or_else(|| "Cannot build without a store.".to_string())?;
// If this beacon chain is being loaded from disk, use the stored head. Otherwise, just use
// the finalized checkpoint (which is probably genesis).
@ -417,17 +423,33 @@ where
.map_err(|e| format!("Unable to init validator pubkey cache: {:?}", e))
})?;
let persisted_fork_choice = store
.get_item::<PersistedForkChoice>(&Hash256::from_slice(&FORK_CHOICE_DB_KEY))
.map_err(|e| format!("DB error when reading persisted fork choice: {:?}", e))?;
let fork_choice = if let Some(persisted) = persisted_fork_choice {
let fc_store =
BeaconForkChoiceStore::from_persisted(persisted.fork_choice_store, store.clone())
.map_err(|e| format!("Unable to load ForkChoiceStore: {:?}", e))?;
ForkChoice::from_persisted(persisted.fork_choice, fc_store)
.map_err(|e| format!("Unable to parse persisted fork choice from disk: {:?}", e))?
} else {
let genesis = &canonical_head;
let fc_store = BeaconForkChoiceStore::get_forkchoice_store(store.clone(), genesis);
ForkChoice::from_genesis(fc_store, &genesis.beacon_block.message)
.map_err(|e| format!("Unable to build initialize ForkChoice: {:?}", e))?
};
let beacon_chain = BeaconChain {
spec: self.spec,
store: self
.store
.ok_or_else(|| "Cannot build without store".to_string())?,
store,
store_migrator: self
.store_migrator
.ok_or_else(|| "Cannot build without store migrator".to_string())?,
slot_clock: self
.slot_clock
.ok_or_else(|| "Cannot build without slot clock".to_string())?,
slot_clock,
op_pool: self
.op_pool
.ok_or_else(|| "Cannot build without op pool".to_string())?,
@ -447,9 +469,7 @@ where
genesis_block_root: self
.genesis_block_root
.ok_or_else(|| "Cannot build without a genesis block root".to_string())?,
fork_choice: self
.fork_choice
.ok_or_else(|| "Cannot build without a fork choice".to_string())?,
fork_choice: RwLock::new(fork_choice),
event_handler: self
.event_handler
.ok_or_else(|| "Cannot build without an event handler".to_string())?,
@ -480,69 +500,6 @@ where
}
}
impl<TStore, TStoreMigrator, TSlotClock, TEth1Backend, TEthSpec, TEventHandler>
BeaconChainBuilder<
Witness<TStore, TStoreMigrator, TSlotClock, TEth1Backend, TEthSpec, TEventHandler>,
>
where
TStore: Store<TEthSpec> + 'static,
TStoreMigrator: Migrate<TEthSpec> + 'static,
TSlotClock: SlotClock + 'static,
TEth1Backend: Eth1ChainBackend<TEthSpec, TStore> + 'static,
TEthSpec: EthSpec + 'static,
TEventHandler: EventHandler<TEthSpec> + 'static,
{
/// Initializes a fork choice with the `ThreadSafeReducedTree` backend.
///
/// If this builder is being "resumed" from disk, then rebuild the last fork choice stored to
/// the database. Otherwise, create a new, empty fork choice.
pub fn reduced_tree_fork_choice(mut self) -> Result<Self, String> {
let store = self
.store
.clone()
.ok_or_else(|| "reduced_tree_fork_choice requires a store.".to_string())?;
let persisted_fork_choice = store
.get_item::<SszForkChoice>(&Hash256::from_slice(&FORK_CHOICE_DB_KEY))
.map_err(|e| format!("DB error when reading persisted fork choice: {:?}", e))?;
let fork_choice = if let Some(persisted) = persisted_fork_choice {
ForkChoice::from_ssz_container(persisted)
.map_err(|e| format!("Unable to read persisted fork choice from disk: {:?}", e))?
} else {
let finalized_snapshot = &self
.finalized_snapshot
.as_ref()
.ok_or_else(|| "reduced_tree_fork_choice requires a finalized_snapshot")?;
let genesis_block_root = self
.genesis_block_root
.ok_or_else(|| "reduced_tree_fork_choice requires a genesis_block_root")?;
let backend = ProtoArrayForkChoice::new(
finalized_snapshot.beacon_block.message.slot,
finalized_snapshot.beacon_block.message.state_root,
// Note: here we set the `justified_epoch` to be the same as the epoch of the
// finalized checkpoint. Whilst this finalized checkpoint may actually point to
// a _later_ justified checkpoint, that checkpoint won't yet exist in the fork
// choice.
finalized_snapshot.beacon_state.current_epoch(),
finalized_snapshot.beacon_state.current_epoch(),
finalized_snapshot.beacon_block_root,
)?;
ForkChoice::new(
backend,
genesis_block_root,
&finalized_snapshot.beacon_state,
)
};
self.fork_choice = Some(fork_choice);
Ok(self)
}
}
impl<TStore, TStoreMigrator, TSlotClock, TEthSpec, TEventHandler>
BeaconChainBuilder<
Witness<
@ -710,8 +667,6 @@ mod test {
.null_event_handler()
.testing_slot_clock(Duration::from_secs(1))
.expect("should configure testing slot clock")
.reduced_tree_fork_choice()
.expect("should add fork choice to builder")
.build()
.expect("should build");

View File

@ -1,5 +1,5 @@
use crate::beacon_chain::ForkChoiceError;
use crate::eth1_chain::Error as Eth1ChainError;
use crate::fork_choice::Error as ForkChoiceError;
use crate::naive_aggregation_pool::Error as NaiveAggregationError;
use crate::observed_attestations::Error as ObservedAttestationsError;
use crate::observed_attesters::Error as ObservedAttestersError;

View File

@ -1,300 +0,0 @@
mod checkpoint_manager;
use crate::{errors::BeaconChainError, metrics, BeaconChain, BeaconChainTypes};
use checkpoint_manager::{get_effective_balances, CheckpointManager, CheckpointWithBalances};
use parking_lot::{RwLock, RwLockReadGuard};
use proto_array_fork_choice::{core::ProtoArray, ProtoArrayForkChoice};
use ssz::{Decode, Encode};
use ssz_derive::{Decode, Encode};
use state_processing::common::get_indexed_attestation;
use std::marker::PhantomData;
use store::{DBColumn, Error as StoreError, StoreItem};
use types::{BeaconBlock, BeaconState, BeaconStateError, Epoch, Hash256, IndexedAttestation, Slot};
type Result<T> = std::result::Result<T, Error>;
#[derive(Debug)]
pub enum Error {
MissingBlock(Hash256),
MissingState(Hash256),
BackendError(String),
BeaconStateError(BeaconStateError),
StoreError(StoreError),
BeaconChainError(Box<BeaconChainError>),
UnknownBlockSlot(Hash256),
UnknownJustifiedBlock(Hash256),
UnknownJustifiedState(Hash256),
UnableToJsonEncode(String),
InvalidAttestation,
}
pub struct ForkChoice<T: BeaconChainTypes> {
backend: ProtoArrayForkChoice,
/// Used for resolving the `0x00..00` alias back to genesis.
///
/// Does not necessarily need to be the _actual_ genesis, it suffices to be the finalized root
/// whenever the struct was instantiated.
genesis_block_root: Hash256,
checkpoint_manager: RwLock<CheckpointManager>,
_phantom: PhantomData<T>,
}
impl<T: BeaconChainTypes> PartialEq for ForkChoice<T> {
/// This implementation ignores the `store`.
fn eq(&self, other: &Self) -> bool {
self.backend == other.backend
&& self.genesis_block_root == other.genesis_block_root
&& *self.checkpoint_manager.read() == *other.checkpoint_manager.read()
}
}
impl<T: BeaconChainTypes> ForkChoice<T> {
/// Instantiate a new fork chooser.
///
/// "Genesis" does not necessarily need to be the absolute genesis, it can be some finalized
/// block.
pub fn new(
backend: ProtoArrayForkChoice,
genesis_block_root: Hash256,
genesis_state: &BeaconState<T::EthSpec>,
) -> Self {
let genesis_checkpoint = CheckpointWithBalances {
epoch: genesis_state.current_epoch(),
root: genesis_block_root,
balances: get_effective_balances(genesis_state),
};
Self {
backend,
genesis_block_root,
checkpoint_manager: RwLock::new(CheckpointManager::new(genesis_checkpoint)),
_phantom: PhantomData,
}
}
/// Run the fork choice rule to determine the head.
pub fn find_head(&self, chain: &BeaconChain<T>) -> Result<Hash256> {
let timer = metrics::start_timer(&metrics::FORK_CHOICE_FIND_HEAD_TIMES);
let remove_alias = |root| {
if root == Hash256::zero() {
self.genesis_block_root
} else {
root
}
};
let mut manager = self.checkpoint_manager.write();
manager.maybe_update(chain.slot()?, chain)?;
let result = self
.backend
.find_head(
manager.current.justified.epoch,
remove_alias(manager.current.justified.root),
manager.current.finalized.epoch,
&manager.current.justified.balances,
)
.map_err(Into::into);
metrics::stop_timer(timer);
result
}
/// Returns true if the given block is known to fork choice.
pub fn contains_block(&self, block_root: &Hash256) -> bool {
self.backend.contains_block(block_root)
}
/// Returns the state root for the given block root.
pub fn block_slot_and_state_root(&self, block_root: &Hash256) -> Option<(Slot, Hash256)> {
self.backend.block_slot_and_state_root(block_root)
}
/// Process all attestations in the given `block`.
///
/// Assumes the block (and therefore its attestations) are valid. It is a logic error to
/// provide an invalid block.
pub fn process_block(
&self,
chain: &BeaconChain<T>,
state: &BeaconState<T::EthSpec>,
block: &BeaconBlock<T::EthSpec>,
block_root: Hash256,
) -> Result<()> {
let timer = metrics::start_timer(&metrics::FORK_CHOICE_PROCESS_BLOCK_TIMES);
self.checkpoint_manager
.write()
.process_state(block_root, state, chain, &self.backend)?;
self.checkpoint_manager
.write()
.maybe_update(chain.slot()?, chain)?;
// Note: we never count the block as a latest message, only attestations.
for attestation in &block.body.attestations {
// If the `data.beacon_block_root` block is not known to the fork choice, simply ignore
// the vote.
if self
.backend
.contains_block(&attestation.data.beacon_block_root)
{
let committee =
state.get_beacon_committee(attestation.data.slot, attestation.data.index)?;
let indexed_attestation =
get_indexed_attestation(committee.committee, &attestation)
.map_err(|_| Error::InvalidAttestation)?;
self.process_indexed_attestation(&indexed_attestation)?;
}
}
// This does not apply a vote to the block, it just makes fork choice aware of the block so
// it can still be identified as the head even if it doesn't have any votes.
self.backend.process_block(
block.slot,
block_root,
block.parent_root,
block.state_root,
state.current_justified_checkpoint.epoch,
state.finalized_checkpoint.epoch,
)?;
metrics::stop_timer(timer);
Ok(())
}
/// Process an attestation which references `block` in `attestation.data.beacon_block_root`.
///
/// Assumes the attestation is valid.
pub fn process_indexed_attestation(
&self,
attestation: &IndexedAttestation<T::EthSpec>,
) -> Result<()> {
let timer = metrics::start_timer(&metrics::FORK_CHOICE_PROCESS_ATTESTATION_TIMES);
let block_hash = attestation.data.beacon_block_root;
// Ignore any attestations to the zero hash.
//
// This is an edge case that results from the spec aliasing the zero hash to the genesis
// block. Attesters may attest to the zero hash if they have never seen a block.
//
// We have two options here:
//
// 1. Apply all zero-hash attestations to the zero hash.
// 2. Ignore all attestations to the zero hash.
//
// (1) becomes weird once we hit finality and fork choice drops the genesis block. (2) is
// fine because votes to the genesis block are not useful; all validators implicitly attest
// to genesis just by being present in the chain.
//
// Additionally, don't add any block hash to fork choice unless we have imported the block.
if block_hash != Hash256::zero() {
for validator_index in attestation.attesting_indices.iter() {
self.backend.process_attestation(
*validator_index as usize,
block_hash,
attestation.data.target.epoch,
)?;
}
}
metrics::stop_timer(timer);
Ok(())
}
/// Returns the latest message for a given validator, if any.
///
/// Returns `(block_root, block_slot)`.
pub fn latest_message(&self, validator_index: usize) -> Option<(Hash256, Epoch)> {
self.backend.latest_message(validator_index)
}
/// Trigger a prune on the underlying fork choice backend.
pub fn prune(&self) -> Result<()> {
let finalized_root = self.checkpoint_manager.read().current.finalized.root;
self.backend.maybe_prune(finalized_root).map_err(Into::into)
}
/// Returns a read-lock to the core `ProtoArray` struct.
///
/// Should only be used when encoding/decoding during troubleshooting.
pub fn core_proto_array(&self) -> RwLockReadGuard<ProtoArray> {
self.backend.core_proto_array()
}
/// Returns a `SszForkChoice` which contains the current state of `Self`.
pub fn as_ssz_container(&self) -> SszForkChoice {
SszForkChoice {
genesis_block_root: self.genesis_block_root.clone(),
checkpoint_manager: self.checkpoint_manager.read().clone(),
backend_bytes: self.backend.as_bytes(),
}
}
/// Instantiates `Self` from a prior `SszForkChoice`.
///
/// The created `Self` will have the same state as the `Self` that created the `SszForkChoice`.
pub fn from_ssz_container(ssz_container: SszForkChoice) -> Result<Self> {
let backend = ProtoArrayForkChoice::from_bytes(&ssz_container.backend_bytes)?;
Ok(Self {
backend,
genesis_block_root: ssz_container.genesis_block_root,
checkpoint_manager: RwLock::new(ssz_container.checkpoint_manager),
_phantom: PhantomData,
})
}
}
/// Helper struct that is used to encode/decode the state of the `ForkChoice` as SSZ bytes.
///
/// This is used when persisting the state of the `BeaconChain` to disk.
#[derive(Encode, Decode, Clone)]
pub struct SszForkChoice {
genesis_block_root: Hash256,
checkpoint_manager: CheckpointManager,
backend_bytes: Vec<u8>,
}
impl From<BeaconStateError> for Error {
fn from(e: BeaconStateError) -> Error {
Error::BeaconStateError(e)
}
}
impl From<BeaconChainError> for Error {
fn from(e: BeaconChainError) -> Error {
Error::BeaconChainError(Box::new(e))
}
}
impl From<StoreError> for Error {
fn from(e: StoreError) -> Error {
Error::StoreError(e)
}
}
impl From<String> for Error {
fn from(e: String) -> Error {
Error::BackendError(e)
}
}
impl StoreItem for SszForkChoice {
fn db_column() -> DBColumn {
DBColumn::ForkChoice
}
fn as_store_bytes(&self) -> Vec<u8> {
self.as_ssz_bytes()
}
fn from_store_bytes(bytes: &[u8]) -> std::result::Result<Self, StoreError> {
Self::from_ssz_bytes(bytes).map_err(Into::into)
}
}

View File

@ -1,340 +0,0 @@
use super::Error;
use crate::{metrics, BeaconChain, BeaconChainTypes};
use proto_array_fork_choice::ProtoArrayForkChoice;
use ssz_derive::{Decode, Encode};
use types::{BeaconState, Checkpoint, Epoch, EthSpec, Hash256, Slot};
const MAX_BALANCE_CACHE_SIZE: usize = 4;
/// An item that is stored in the `BalancesCache`.
#[derive(PartialEq, Clone, Encode, Decode)]
struct CacheItem {
/// The block root at which `self.balances` are valid.
block_root: Hash256,
/// The `state.balances` list.
balances: Vec<u64>,
}
/// Provides a cache to avoid reading `BeaconState` from disk when updating the current justified
/// checkpoint.
///
/// It should store a mapping of `epoch_boundary_block_root -> state.balances`.
#[derive(PartialEq, Clone, Default, Encode, Decode)]
struct BalancesCache {
items: Vec<CacheItem>,
}
impl BalancesCache {
/// Inspect the given `state` and determine the root of the block at the first slot of
/// `state.current_epoch`. If there is not already some entry for the given block root, then
/// add `state.balances` to the cache.
pub fn process_state<E: EthSpec>(
&mut self,
block_root: Hash256,
state: &BeaconState<E>,
) -> Result<(), Error> {
// We are only interested in balances from states that are at the start of an epoch,
// because this is where the `current_justified_checkpoint.root` will point.
if !Self::is_first_block_in_epoch(block_root, state)? {
return Ok(());
}
let epoch_boundary_slot = state.current_epoch().start_slot(E::slots_per_epoch());
let epoch_boundary_root = if epoch_boundary_slot == state.slot {
block_root
} else {
// This call remains sensible as long as `state.block_roots` is larger than a single
// epoch.
*state.get_block_root(epoch_boundary_slot)?
};
if self.position(epoch_boundary_root).is_none() {
let item = CacheItem {
block_root: epoch_boundary_root,
balances: get_effective_balances(state),
};
if self.items.len() == MAX_BALANCE_CACHE_SIZE {
self.items.remove(0);
}
self.items.push(item);
}
Ok(())
}
/// Returns `true` if the given `block_root` is the first/only block to have been processed in
/// the epoch of the given `state`.
///
/// We can determine if it is the first block by looking back through `state.block_roots` to
/// see if there is a block in the current epoch with a different root.
fn is_first_block_in_epoch<E: EthSpec>(
block_root: Hash256,
state: &BeaconState<E>,
) -> Result<bool, Error> {
let mut prior_block_found = false;
for slot in state.current_epoch().slot_iter(E::slots_per_epoch()) {
if slot < state.slot {
if *state.get_block_root(slot)? != block_root {
prior_block_found = true;
break;
}
} else {
break;
}
}
Ok(!prior_block_found)
}
fn position(&self, block_root: Hash256) -> Option<usize> {
self.items
.iter()
.position(|item| item.block_root == block_root)
}
/// Get the balances for the given `block_root`, if any.
///
/// If some balances are found, they are removed from the cache.
pub fn get(&mut self, block_root: Hash256) -> Option<Vec<u64>> {
let i = self.position(block_root)?;
Some(self.items.remove(i).balances)
}
}
/// Returns the effective balances for every validator in the given `state`.
///
/// Any validator who is not active in the epoch of the given `state` is assigned a balance of
/// zero.
pub fn get_effective_balances<T: EthSpec>(state: &BeaconState<T>) -> Vec<u64> {
state
.validators
.iter()
.map(|validator| {
if validator.is_active_at(state.current_epoch()) {
validator.effective_balance
} else {
0
}
})
.collect()
}
/// A `types::Checkpoint` that also stores the validator balances from a `BeaconState`.
///
/// Useful because we need to track the justified checkpoint balances.
#[derive(PartialEq, Clone, Encode, Decode)]
pub struct CheckpointWithBalances {
pub epoch: Epoch,
pub root: Hash256,
/// These are the balances of the state with `self.root`.
///
/// Importantly, these are _not_ the balances of the first state that we saw that has
/// `self.epoch` and `self.root` as `state.current_justified_checkpoint`. These are the
/// balances of the state from the block with `state.current_justified_checkpoint.root`.
pub balances: Vec<u64>,
}
impl Into<Checkpoint> for CheckpointWithBalances {
fn into(self) -> Checkpoint {
Checkpoint {
epoch: self.epoch,
root: self.root,
}
}
}
/// A pair of checkpoints, representing `state.current_justified_checkpoint` and
/// `state.finalized_checkpoint` for some `BeaconState`.
#[derive(PartialEq, Clone, Encode, Decode)]
pub struct FFGCheckpoints {
pub justified: CheckpointWithBalances,
pub finalized: Checkpoint,
}
/// A struct to manage the justified and finalized checkpoints to be used for `ForkChoice`.
///
/// This struct exists to manage the `should_update_justified_checkpoint` logic in the fork choice
/// section of the spec:
///
/// https://github.com/ethereum/eth2.0-specs/blob/dev/specs/phase0/fork-choice.md#should_update_justified_checkpoint
#[derive(PartialEq, Clone, Encode, Decode)]
pub struct CheckpointManager {
/// The current FFG checkpoints that should be used for finding the head.
pub current: FFGCheckpoints,
/// The best-known checkpoints that should be moved to `self.current` when the time is right.
best: FFGCheckpoints,
/// The epoch at which `self.current` should become `self.best`, if any.
update_at: Option<Epoch>,
/// A cached used to try and avoid DB reads when updating `self.current` and `self.best`.
balances_cache: BalancesCache,
}
impl CheckpointManager {
/// Create a new checkpoint cache from `genesis_checkpoint` derived from the genesis block.
pub fn new(genesis_checkpoint: CheckpointWithBalances) -> Self {
let ffg_checkpoint = FFGCheckpoints {
justified: genesis_checkpoint.clone(),
finalized: genesis_checkpoint.into(),
};
Self {
current: ffg_checkpoint.clone(),
best: ffg_checkpoint,
update_at: None,
balances_cache: BalancesCache::default(),
}
}
/// Potentially updates `self.current`, if the conditions are correct.
///
/// Should be called before running the fork choice `find_head` function to ensure
/// `self.current` is up-to-date.
pub fn maybe_update<T: BeaconChainTypes>(
&mut self,
current_slot: Slot,
chain: &BeaconChain<T>,
) -> Result<(), Error> {
if self.best.justified.epoch > self.current.justified.epoch {
let current_epoch = current_slot.epoch(T::EthSpec::slots_per_epoch());
match self.update_at {
None => {
if self.best.justified.epoch > self.current.justified.epoch {
if Self::compute_slots_since_epoch_start::<T>(current_slot)
< chain.spec.safe_slots_to_update_justified
{
self.current = self.best.clone();
} else {
self.update_at = Some(current_epoch + 1)
}
}
}
Some(epoch) if epoch <= current_epoch => {
self.current = self.best.clone();
self.update_at = None
}
_ => {}
}
}
Ok(())
}
/// Checks the given `state` (must correspond to the given `block_root`) to see if it contains
/// a `current_justified_checkpoint` that is better than `self.best_justified_checkpoint`. If
/// so, the value is updated.
///
/// Note: this does not update `self.justified_checkpoint`.
pub fn process_state<T: BeaconChainTypes>(
&mut self,
block_root: Hash256,
state: &BeaconState<T::EthSpec>,
chain: &BeaconChain<T>,
proto_array: &ProtoArrayForkChoice,
) -> Result<(), Error> {
// Only proceed if the new checkpoint is better than our current checkpoint.
if state.current_justified_checkpoint.epoch > self.current.justified.epoch
&& state.finalized_checkpoint.epoch >= self.current.finalized.epoch
{
let candidate = FFGCheckpoints {
justified: CheckpointWithBalances {
epoch: state.current_justified_checkpoint.epoch,
root: state.current_justified_checkpoint.root,
balances: self
.get_balances_for_block(state.current_justified_checkpoint.root, chain)?,
},
finalized: state.finalized_checkpoint.clone(),
};
// Using the given `state`, determine its ancestor at the slot of our current justified
// epoch. Later, this will be compared to the root of the current justified checkpoint
// to determine if this state is descendant of our current justified state.
let new_checkpoint_ancestor = Self::get_block_root_at_slot(
state,
chain,
candidate.justified.root,
self.current
.justified
.epoch
.start_slot(T::EthSpec::slots_per_epoch()),
)?;
let candidate_justified_block_slot = proto_array
.block_slot(&candidate.justified.root)
.ok_or_else(|| Error::UnknownBlockSlot(candidate.justified.root))?;
// If the new justified checkpoint is an ancestor of the current justified checkpoint,
// it is always safe to change it.
if new_checkpoint_ancestor == Some(self.current.justified.root)
&& candidate_justified_block_slot
>= candidate
.justified
.epoch
.start_slot(T::EthSpec::slots_per_epoch())
{
self.current = candidate.clone()
}
if candidate.justified.epoch > self.best.justified.epoch {
// Always update the best checkpoint, if it's better.
self.best = candidate;
}
// Add the state's balances to the balances cache to avoid a state read later.
self.balances_cache.process_state(block_root, state)?;
}
Ok(())
}
fn get_balances_for_block<T: BeaconChainTypes>(
&mut self,
block_root: Hash256,
chain: &BeaconChain<T>,
) -> Result<Vec<u64>, Error> {
if let Some(balances) = self.balances_cache.get(block_root) {
metrics::inc_counter(&metrics::BALANCES_CACHE_HITS);
Ok(balances)
} else {
metrics::inc_counter(&metrics::BALANCES_CACHE_MISSES);
let block = chain
.get_block(&block_root)?
.ok_or_else(|| Error::UnknownJustifiedBlock(block_root))?;
let state = chain
.get_state(&block.state_root(), Some(block.slot()))?
.ok_or_else(|| Error::UnknownJustifiedState(block.state_root()))?;
Ok(get_effective_balances(&state))
}
}
/// Attempts to get the block root for the given `slot`.
///
/// First, the `state` is used to see if the slot is within the distance of its historical
/// lists. Then, the `chain` is used which will anchor the search at the given
/// `justified_root`.
fn get_block_root_at_slot<T: BeaconChainTypes>(
state: &BeaconState<T::EthSpec>,
chain: &BeaconChain<T>,
justified_root: Hash256,
slot: Slot,
) -> Result<Option<Hash256>, Error> {
match state.get_block_root(slot) {
Ok(root) => Ok(Some(*root)),
Err(_) => chain
.get_ancestor_block_root(justified_root, slot)
.map_err(Into::into),
}
}
/// Calculate how far `slot` lies from the start of its epoch.
fn compute_slots_since_epoch_start<T: BeaconChainTypes>(slot: Slot) -> u64 {
let slots_per_epoch = T::EthSpec::slots_per_epoch();
(slot - slot.epoch(slots_per_epoch).start_slot(slots_per_epoch)).as_u64()
}
}

View File

@ -4,13 +4,13 @@ extern crate lazy_static;
pub mod attestation_verification;
mod beacon_chain;
mod beacon_fork_choice_store;
mod beacon_snapshot;
mod block_verification;
pub mod builder;
mod errors;
pub mod eth1_chain;
pub mod events;
mod fork_choice;
mod head_tracker;
mod metrics;
pub mod migrate;
@ -19,6 +19,7 @@ mod observed_attestations;
mod observed_attesters;
mod observed_block_producers;
mod persisted_beacon_chain;
mod persisted_fork_choice;
mod shuffling_cache;
mod snapshot_cache;
pub mod test_utils;
@ -27,15 +28,15 @@ mod validator_pubkey_cache;
pub use self::beacon_chain::{
AttestationProcessingOutcome, BeaconChain, BeaconChainTypes, ChainSegmentResult,
StateSkipConfig,
ForkChoiceError, StateSkipConfig,
};
pub use self::beacon_snapshot::BeaconSnapshot;
pub use self::errors::{BeaconChainError, BlockProductionError};
pub use attestation_verification::Error as AttestationError;
pub use beacon_fork_choice_store::{BeaconForkChoiceStore, Error as ForkChoiceStoreError};
pub use block_verification::{BlockError, BlockProcessingOutcome, GossipVerifiedBlock};
pub use eth1_chain::{Eth1Chain, Eth1ChainBackend};
pub use events::EventHandler;
pub use fork_choice::ForkChoice;
pub use metrics::scrape_for_metrics;
pub use parking_lot;
pub use slot_clock;

View File

@ -49,10 +49,6 @@ lazy_static! {
"beacon_block_processing_db_write_seconds",
"Time spent writing a newly processed block and state to DB"
);
pub static ref BLOCK_PROCESSING_FORK_CHOICE_REGISTER: Result<Histogram> = try_create_histogram(
"beacon_block_processing_fork_choice_register_seconds",
"Time spent registering the new block with fork choice (but not finding head)"
);
pub static ref BLOCK_PROCESSING_ATTESTATION_OBSERVATION: Result<Histogram> = try_create_histogram(
"beacon_block_processing_attestation_observation_seconds",
"Time spent hashing and remembering all the attestations in the block"
@ -115,10 +111,6 @@ lazy_static! {
/*
* General Attestation Processing
*/
pub static ref ATTESTATION_PROCESSING_APPLY_TO_FORK_CHOICE: Result<Histogram> = try_create_histogram(
"beacon_attestation_processing_apply_to_fork_choice",
"Time spent applying an attestation to fork choice"
);
pub static ref ATTESTATION_PROCESSING_APPLY_TO_AGG_POOL: Result<Histogram> = try_create_histogram(
"beacon_attestation_processing_apply_to_agg_pool",
"Time spent applying an attestation to the naive aggregation pool"

View File

@ -0,0 +1,25 @@
use crate::beacon_fork_choice_store::PersistedForkChoiceStore as ForkChoiceStore;
use fork_choice::PersistedForkChoice as ForkChoice;
use ssz::{Decode, Encode};
use ssz_derive::{Decode, Encode};
use store::{DBColumn, Error, StoreItem};
#[derive(Encode, Decode)]
pub struct PersistedForkChoice {
pub fork_choice: ForkChoice,
pub fork_choice_store: ForkChoiceStore,
}
impl StoreItem for PersistedForkChoice {
fn db_column() -> DBColumn {
DBColumn::ForkChoice
}
fn as_store_bytes(&self) -> Vec<u8> {
self.as_ssz_bytes()
}
fn from_store_bytes(bytes: &[u8]) -> std::result::Result<Self, Error> {
Self::from_ssz_bytes(bytes).map_err(Into::into)
}
}

View File

@ -122,8 +122,6 @@ impl<E: EthSpec> BeaconChainHarness<HarnessType<E>> {
.null_event_handler()
.testing_slot_clock(HARNESS_SLOT_TIME)
.expect("should configure testing slot clock")
.reduced_tree_fork_choice()
.expect("should add fork choice to builder")
.build()
.expect("should build");
@ -164,8 +162,6 @@ impl<E: EthSpec> BeaconChainHarness<DiskHarnessType<E>> {
.null_event_handler()
.testing_slot_clock(HARNESS_SLOT_TIME)
.expect("should configure testing slot clock")
.reduced_tree_fork_choice()
.expect("should add fork choice to builder")
.build()
.expect("should build");
@ -201,8 +197,6 @@ impl<E: EthSpec> BeaconChainHarness<DiskHarnessType<E>> {
.null_event_handler()
.testing_slot_clock(Duration::from_secs(1))
.expect("should configure testing slot clock")
.reduced_tree_fork_choice()
.expect("should add fork choice to builder")
.build()
.expect("should build");
@ -243,6 +237,35 @@ where
block_strategy: BlockStrategy,
attestation_strategy: AttestationStrategy,
) -> Hash256 {
let mut i = 0;
self.extend_chain_while(
|_, _| {
i += 1;
i <= num_blocks
},
block_strategy,
attestation_strategy,
)
}
/// Extend the `BeaconChain` with some blocks and attestations. Returns the root of the
/// last-produced block (the head of the chain).
///
/// Chain will be extended while `predidcate` returns `true`.
///
/// The `block_strategy` dictates where the new blocks will be placed.
///
/// The `attestation_strategy` dictates which validators will attest to the newly created
/// blocks.
pub fn extend_chain_while<F>(
&self,
mut predicate: F,
block_strategy: BlockStrategy,
attestation_strategy: AttestationStrategy,
) -> Hash256
where
F: FnMut(&SignedBeaconBlock<E>, &BeaconState<E>) -> bool,
{
let mut state = {
// Determine the slot for the first block (or skipped block).
let state_slot = match block_strategy {
@ -265,13 +288,17 @@ where
let mut head_block_root = None;
for _ in 0..num_blocks {
loop {
let (block, new_state) = self.build_block(state.clone(), slot, block_strategy);
if !predicate(&block, &new_state) {
break;
}
while self.chain.slot().expect("should have a slot") < slot {
self.advance_slot();
}
let (block, new_state) = self.build_block(state.clone(), slot, block_strategy);
let block_root = self
.chain
.process_block(block)
@ -289,6 +316,39 @@ where
head_block_root.expect("did not produce any blocks")
}
/// A simple method to produce a block at the current slot without applying it to the chain.
///
/// Always uses `BlockStrategy::OnCanonicalHead`.
pub fn get_block(&self) -> (SignedBeaconBlock<E>, BeaconState<E>) {
let state = self
.chain
.state_at_slot(
self.chain.slot().unwrap() - 1,
StateSkipConfig::WithStateRoots,
)
.unwrap();
let slot = self.chain.slot().unwrap();
self.build_block(state, slot, BlockStrategy::OnCanonicalHead)
}
/// A simple method to produce and process all attestation at the current slot. Always uses
/// `AttestationStrategy::AllValidators`.
pub fn generate_all_attestations(&self) {
let slot = self.chain.slot().unwrap();
let (state, block_root) = {
let head = self.chain.head().unwrap();
(head.beacon_state.clone(), head.beacon_block_root)
};
self.add_attestations_for_slot(
&AttestationStrategy::AllValidators,
&state,
block_root,
slot,
);
}
/// Returns current canonical head slot
pub fn get_chain_slot(&self) -> Slot {
self.chain.slot().unwrap()
@ -626,14 +686,16 @@ where
spec,
);
self.chain
let attn = self.chain
.verify_aggregated_attestation_for_gossip(signed_aggregate)
.expect("should not error during attestation processing")
.add_to_pool(&self.chain)
.expect("should add attestation to naive aggregation pool")
.add_to_fork_choice(&self.chain)
.expect("should not error during attestation processing");
self.chain.apply_attestation_to_fork_choice(&attn)
.expect("should add attestation to fork choice");
}
self.chain.add_to_block_inclusion_pool(attn)
.expect("should add attestation to op pool");
}
});
}

View File

@ -721,244 +721,6 @@ fn unaggregated_gossip_verification() {
);
}
/// Tests the verification conditions for an unaggregated attestation on the gossip network.
#[test]
fn fork_choice_verification() {
let harness = get_harness(VALIDATOR_COUNT);
let chain = &harness.chain;
// Extend the chain out a few epochs so we have some chain depth to play with.
harness.extend_chain(
MainnetEthSpec::slots_per_epoch() as usize * 3 - 1,
BlockStrategy::OnCanonicalHead,
AttestationStrategy::AllValidators,
);
// Advance into a slot where there have not been blocks or attestations produced.
harness.advance_slot();
// We're going to produce the attestations at the first slot of the epoch.
let (valid_attestation, _validator_index, _validator_committee_index, _validator_sk) =
get_valid_unaggregated_attestation(&harness.chain);
// Extend the chain two more blocks, but without any attestations so we don't trigger the
// "already seen" caches.
//
// Because of this, the attestation we're dealing with was made one slot prior to the current
// slot. This allows us to test the `AttestsToFutureBlock` condition.
harness.extend_chain(
2,
BlockStrategy::OnCanonicalHead,
AttestationStrategy::SomeValidators(vec![]),
);
let current_slot = chain.slot().expect("should get slot");
let expected_current_epoch = chain.epoch().expect("should get epoch");
let attestation = harness
.chain
.verify_unaggregated_attestation_for_gossip(valid_attestation.clone())
.expect("precondition: should gossip verify attestation");
macro_rules! assert_invalid {
($desc: tt, $attn_getter: expr, $($error: pat) |+ $( if $guard: expr )?) => {
assert!(
matches!(
harness
.chain
.apply_attestation_to_fork_choice(&$attn_getter)
.err()
.expect(&format!(
"{} should error during apply_attestation_to_fork_choice",
$desc
)),
$( $error ) |+ $( if $guard )?
),
"case: {}",
$desc,
);
};
}
assert_invalid!(
"attestation without any aggregation bits set",
{
let mut a = attestation.clone();
a.__indexed_attestation_mut().attesting_indices = vec![].into();
a
},
AttnError::EmptyAggregationBitfield
);
/*
* The following two tests ensure that:
*
* Spec v0.11.2
*
* assert target.epoch in [expected_current_epoch, previous_epoch]
*/
let future_epoch = expected_current_epoch + 1;
assert_invalid!(
"attestation from future epoch",
{
let mut a = attestation.clone();
a.__indexed_attestation_mut().data.target.epoch = future_epoch;
a
},
AttnError::FutureEpoch {
attestation_epoch,
current_epoch,
}
if attestation_epoch == future_epoch && current_epoch == expected_current_epoch
);
assert!(
expected_current_epoch > 1,
"precondition: must be able to have a past epoch"
);
let past_epoch = expected_current_epoch - 2;
assert_invalid!(
"attestation from past epoch",
{
let mut a = attestation.clone();
a.__indexed_attestation_mut().data.target.epoch = past_epoch;
a
},
AttnError::PastEpoch {
attestation_epoch,
current_epoch,
}
if attestation_epoch == past_epoch && current_epoch == expected_current_epoch
);
/*
* This test ensures that:
*
* Spec v0.11.2
*
* assert target.epoch == compute_epoch_at_slot(attestation.data.slot)
*/
assert_invalid!(
"attestation with bad target epoch",
{
let mut a = attestation.clone();
let indexed = a.__indexed_attestation_mut();
indexed.data.target.epoch = indexed.data.slot.epoch(E::slots_per_epoch()) - 1;
a
},
AttnError::BadTargetEpoch
);
/*
* This test ensures that:
*
* Spec v0.11.2
*
* Attestations target be for a known block. If target block is unknown, delay consideration
* until the block is found
*
* assert target.root in store.blocks
*/
let unknown_root = Hash256::from_low_u64_le(42);
assert_invalid!(
"attestation with unknown target root",
{
let mut a = attestation.clone();
let indexed = a.__indexed_attestation_mut();
indexed.data.target.root = unknown_root;
a
},
AttnError::UnknownTargetRoot(hash) if hash == unknown_root
);
// NOTE: we're not testing an assert from the spec:
//
// `assert get_current_slot(store) >= compute_start_slot_at_epoch(target.epoch)`
//
// I think this check is redundant and I've raised an issue here:
//
// https://github.com/ethereum/eth2.0-specs/pull/1755
/*
* This test asserts that:
*
* Spec v0.11.2
*
* # Attestations must be for a known block. If block is unknown, delay consideration until the
* block is found
*
* assert attestation.data.beacon_block_root in store.blocks
*/
assert_invalid!(
"attestation with unknown beacon block root",
{
let mut a = attestation.clone();
let indexed = a.__indexed_attestation_mut();
indexed.data.beacon_block_root = unknown_root;
a
},
AttnError::UnknownHeadBlock {
beacon_block_root
}
if beacon_block_root == unknown_root
);
let future_block = harness
.chain
.block_at_slot(current_slot)
.expect("should not error getting block")
.expect("should find block at current slot");
assert_invalid!(
"attestation to future block",
{
let mut a = attestation.clone();
let indexed = a.__indexed_attestation_mut();
assert!(
future_block.slot() > indexed.data.slot,
"precondition: the attestation must attest to the future"
);
indexed.data.beacon_block_root = future_block.canonical_root();
a
},
AttnError::AttestsToFutureBlock {
block: current_slot,
attestation: slot,
}
if slot == current_slot - 1
);
// Note: we're not checking the "attestations can only affect the fork choice of subsequent
// slots" part of the spec, we do this upstream.
assert!(
harness
.chain
.apply_attestation_to_fork_choice(&attestation.clone())
.is_ok(),
"should verify valid attestation"
);
// There's nothing stopping fork choice from accepting the same attestation twice.
assert!(
harness
.chain
.apply_attestation_to_fork_choice(&attestation)
.is_ok(),
"should verify valid attestation a second time"
);
}
/// Ensures that an attestation that skips epochs can still be processed.
///
/// This also checks that we can do a state lookup if we don't get a hit from the shuffling cache.

View File

@ -154,7 +154,7 @@ fn assert_chains_pretty_much_the_same<T: BeaconChainTypes>(a: &BeaconChain<T>, b
"genesis_block_root should be equal"
);
assert!(
a.fork_choice == b.fork_choice,
*a.fork_choice.read() == *b.fork_choice.read(),
"fork_choice should be equal"
);
}

View File

@ -375,7 +375,13 @@ fn unaggregated_attestations_added_to_fork_choice_some_none() {
);
let state = &harness.chain.head().expect("should get head").beacon_state;
let fork_choice = &harness.chain.fork_choice;
let mut fork_choice = harness.chain.fork_choice.write();
// Move forward a slot so all queued attestations can be processed.
harness.advance_slot();
fork_choice
.update_time(harness.chain.slot().unwrap())
.unwrap();
let validator_slots: Vec<(usize, Slot)> = (0..VALIDATOR_COUNT)
.into_iter()
@ -397,7 +403,7 @@ fn unaggregated_attestations_added_to_fork_choice_some_none() {
assert_eq!(
latest_message.unwrap().1,
slot.epoch(MinimalEthSpec::slots_per_epoch()),
"Latest message slot for {} should be equal to slot {}.",
"Latest message epoch for {} should be equal to epoch {}.",
validator,
slot
)
@ -485,7 +491,13 @@ fn unaggregated_attestations_added_to_fork_choice_all_updated() {
);
let state = &harness.chain.head().expect("should get head").beacon_state;
let fork_choice = &harness.chain.fork_choice;
let mut fork_choice = harness.chain.fork_choice.write();
// Move forward a slot so all queued attestations can be processed.
harness.advance_slot();
fork_choice
.update_time(harness.chain.slot().unwrap())
.unwrap();
let validators: Vec<usize> = (0..VALIDATOR_COUNT).collect();
let slots: Vec<Slot> = validators

View File

@ -387,8 +387,6 @@ where
.clone()
.ok_or_else(|| "beacon_chain requires a slot clock")?,
)
.reduced_tree_fork_choice()
.map_err(|e| format!("Failed to init fork choice: {}", e))?
.build()
.map_err(|e| format!("Failed to build beacon chain: {}", e))?;

View File

@ -64,8 +64,6 @@ mod tests {
Duration::from_secs(recent_genesis_time()),
Duration::from_millis(SLOT_DURATION_MILLIS),
))
.reduced_tree_fork_choice()
.expect("should add fork choice to builder")
.build()
.expect("should build"),
);

View File

@ -2,10 +2,11 @@ use crate::service::NetworkMessage;
use crate::sync::{PeerSyncInfo, SyncMessage};
use beacon_chain::{
attestation_verification::{
Error as AttnError, IntoForkChoiceVerifiedAttestation, VerifiedAggregatedAttestation,
Error as AttnError, SignatureVerifiedAttestation, VerifiedAggregatedAttestation,
VerifiedUnaggregatedAttestation,
},
BeaconChain, BeaconChainTypes, BlockError, BlockProcessingOutcome, GossipVerifiedBlock,
BeaconChain, BeaconChainError, BeaconChainTypes, BlockError, BlockProcessingOutcome,
ForkChoiceError, GossipVerifiedBlock,
};
use eth2_libp2p::rpc::*;
use eth2_libp2p::{NetworkGlobals, PeerId, Request, Response};
@ -881,16 +882,27 @@ impl<T: BeaconChainTypes> Processor<T> {
&self,
peer_id: PeerId,
beacon_block_root: Hash256,
attestation: &'a impl IntoForkChoiceVerifiedAttestation<'a, T>,
attestation: &'a impl SignatureVerifiedAttestation<T>,
) {
if let Err(e) = self.chain.apply_attestation_to_fork_choice(attestation) {
debug!(
self.log,
"Attestation invalid for fork choice";
"reason" => format!("{:?}", e),
"peer" => format!("{:?}", peer_id),
"beacon_block_root" => format!("{:?}", beacon_block_root)
)
match e {
BeaconChainError::ForkChoiceError(ForkChoiceError::InvalidAttestation(e)) => {
debug!(
self.log,
"Attestation invalid for fork choice";
"reason" => format!("{:?}", e),
"peer" => format!("{:?}", peer_id),
"beacon_block_root" => format!("{:?}", beacon_block_root)
)
}
e => error!(
self.log,
"Error applying attestation to fork choice";
"reason" => format!("{:?}", e),
"peer" => format!("{:?}", peer_id),
"beacon_block_root" => format!("{:?}", beacon_block_root)
),
}
}
}
}

View File

@ -269,7 +269,12 @@ impl<T: BeaconChainTypes> SyncManager<T> {
// consider it synced (it can be the case that the peer seems ahead of us, but we
// reject its chain).
if self.chain.fork_choice.contains_block(&remote.head_root) {
if self
.chain
.fork_choice
.read()
.contains_block(&remote.head_root)
{
self.synced_peer(&peer_id, remote);
// notify the range sync that a peer has been added
self.range_sync.fully_synced_peer_found();

View File

@ -455,6 +455,7 @@ impl<T: BeaconChainTypes> ChainCollection<T> {
if chain.target_head_slot <= local_finalized_slot
|| beacon_chain
.fork_choice
.read()
.contains_block(&chain.target_head_root)
{
debug!(log_ref, "Purging out of finalized chain"; "start_epoch" => chain.start_epoch, "end_slot" => chain.target_head_slot);
@ -468,6 +469,7 @@ impl<T: BeaconChainTypes> ChainCollection<T> {
if chain.target_head_slot <= local_finalized_slot
|| beacon_chain
.fork_choice
.read()
.contains_block(&chain.target_head_root)
{
debug!(log_ref, "Purging out of date head chain"; "start_epoch" => chain.start_epoch, "end_slot" => chain.target_head_slot);

View File

@ -30,6 +30,7 @@ impl RangeSyncType {
if remote_info.finalized_epoch > local_info.finalized_epoch
&& !chain
.fork_choice
.read()
.contains_block(&remote_info.finalized_root)
{
RangeSyncType::Finalized

View File

@ -12,7 +12,13 @@ pub fn get_fork_choice<T: BeaconChainTypes>(
req: Request<Body>,
beacon_chain: Arc<BeaconChain<T>>,
) -> ApiResult {
ResponseBuilder::new(&req)?.body_no_ssz(&*beacon_chain.fork_choice.core_proto_array())
ResponseBuilder::new(&req)?.body_no_ssz(
&*beacon_chain
.fork_choice
.read()
.proto_array()
.core_proto_array(),
)
}
/// Returns the `PersistedOperationPool` struct.

View File

@ -2,8 +2,8 @@ use crate::helpers::{check_content_type_for_json, publish_beacon_block_to_networ
use crate::response_builder::ResponseBuilder;
use crate::{ApiError, ApiResult, NetworkChannel, UrlQuery};
use beacon_chain::{
attestation_verification::Error as AttnError, BeaconChain, BeaconChainTypes, BlockError,
StateSkipConfig,
attestation_verification::Error as AttnError, BeaconChain, BeaconChainError, BeaconChainTypes,
BlockError, ForkChoiceError, StateSkipConfig,
};
use bls::PublicKeyBytes;
use eth2_libp2p::PubsubMessage;
@ -506,7 +506,7 @@ fn process_unaggregated_attestation<T: BeaconChainTypes>(
beacon_chain
.apply_attestation_to_fork_choice(&verified_attestation)
.map_err(|e| {
handle_attestation_error(
handle_fork_choice_error(
e,
&format!(
"unaggregated attestation {} was unable to be added to fork choice",
@ -645,7 +645,7 @@ fn process_aggregated_attestation<T: BeaconChainTypes>(
beacon_chain
.apply_attestation_to_fork_choice(&verified_attestation)
.map_err(|e| {
handle_attestation_error(
handle_fork_choice_error(
e,
&format!(
"aggregated attestation {} was unable to be added to fork choice",
@ -717,3 +717,48 @@ fn handle_attestation_error(
}
}
}
/// Common handler for `ForkChoiceError` during attestation verification.
fn handle_fork_choice_error(
e: BeaconChainError,
detail: &str,
data: &AttestationData,
log: &Logger,
) -> ApiError {
match e {
BeaconChainError::ForkChoiceError(ForkChoiceError::InvalidAttestation(e)) => {
error!(
log,
"Local attestation invalid for fork choice";
"detail" => detail,
"reason" => format!("{:?}", e),
"target" => data.target.epoch,
"source" => data.source.epoch,
"index" => data.index,
"slot" => data.slot,
);
ApiError::ProcessingError(format!(
"Invalid local attestation. Error: {:?} Detail: {}",
e, detail
))
}
e => {
error!(
log,
"Internal error applying attn to fork choice";
"detail" => detail,
"error" => format!("{:?}", e),
"target" => data.target.epoch,
"source" => data.source.epoch,
"index" => data.index,
"slot" => data.slot,
);
ApiError::ServerError(format!(
"Internal error verifying local attestation. Error: {:?}. Detail: {}",
e, detail
))
}
}
}

View File

@ -14,6 +14,7 @@ use remote_beacon_node::{
use rest_types::ValidatorDutyBytes;
use std::convert::TryInto;
use std::sync::Arc;
use std::time::{SystemTime, UNIX_EPOCH};
use types::{
test_utils::{
build_double_vote_attester_slashing, build_proposer_slashing,
@ -410,10 +411,16 @@ fn validator_block_post() {
let spec = &E::default_spec();
let two_slots_secs = (spec.milliseconds_per_slot / 1_000) * 2;
let mut config = testing_client_config();
config.genesis = ClientGenesis::Interop {
validator_count: 8,
genesis_time: 13_371_337,
genesis_time: SystemTime::now()
.duration_since(UNIX_EPOCH)
.unwrap()
.as_secs()
- two_slots_secs,
};
let node = build_node(&mut env, config);
@ -935,6 +942,8 @@ fn get_fork_choice() {
.beacon_chain()
.expect("node should have beacon chain")
.fork_choice
.read()
.proto_array()
.core_proto_array(),
"result should be as expected"
);

View File

@ -30,3 +30,4 @@ serde_derive = "1.0.110"
lazy_static = "1.4.0"
lighthouse_metrics = { path = "../../common/lighthouse_metrics" }
lru = "0.5.1"
fork_choice = { path = "../../consensus/fork_choice" }

View File

@ -17,5 +17,5 @@ hex = "0.4.2"
eth2_ssz = "0.1.2"
serde_json = "1.0.52"
eth2_config = { path = "../eth2_config" }
proto_array_fork_choice = { path = "../../consensus/proto_array_fork_choice" }
proto_array = { path = "../../consensus/proto_array" }
operation_pool = { path = "../../beacon_node/operation_pool" }

View File

@ -17,7 +17,7 @@ use types::{
use url::Url;
pub use operation_pool::PersistedOperationPool;
pub use proto_array_fork_choice::core::ProtoArray;
pub use proto_array::core::ProtoArray;
pub use rest_types::{
CanonicalHeadResponse, Committee, HeadBeaconBlock, Health, IndividualVotesRequest,
IndividualVotesResponse, SyncingResponse, ValidatorDutiesRequest, ValidatorDutyBytes,

View File

@ -24,6 +24,11 @@ pub trait SlotClock: Send + Sync + Sized {
/// Returns the slot at this present time.
fn now(&self) -> Option<Slot>;
/// Indicates if the current time is prior to genesis time.
///
/// Returns `None` if the system clock cannot be read.
fn is_prior_to_genesis(&self) -> Option<bool>;
/// Returns the present time as a duration since the UNIX epoch.
///
/// Returns `None` if the present time is before the UNIX epoch (unlikely).

View File

@ -41,6 +41,10 @@ impl ManualSlotClock {
self.set_slot(self.now().unwrap().as_u64() + 1)
}
pub fn genesis_duration(&self) -> &Duration {
&self.genesis_duration
}
/// Returns the duration between UNIX epoch and the start of `slot`.
pub fn start_of(&self, slot: Slot) -> Option<Duration> {
let slot = slot
@ -104,6 +108,10 @@ impl SlotClock for ManualSlotClock {
self.slot_of(*self.current_time.read())
}
fn is_prior_to_genesis(&self) -> Option<bool> {
Some(*self.current_time.read() < self.genesis_duration)
}
fn now_duration(&self) -> Option<Duration> {
Some(*self.current_time.read())
}
@ -160,6 +168,26 @@ mod tests {
assert_eq!(clock.now(), Some(Slot::new(123)));
}
#[test]
fn test_is_prior_to_genesis() {
let genesis_secs = 1;
let clock = ManualSlotClock::new(
Slot::new(0),
Duration::from_secs(genesis_secs),
Duration::from_secs(1),
);
*clock.current_time.write() = Duration::from_secs(genesis_secs - 1);
assert!(clock.is_prior_to_genesis().unwrap(), "prior to genesis");
*clock.current_time.write() = Duration::from_secs(genesis_secs);
assert!(!clock.is_prior_to_genesis().unwrap(), "at genesis");
*clock.current_time.write() = Duration::from_secs(genesis_secs + 1);
assert!(!clock.is_prior_to_genesis().unwrap(), "after genesis");
}
#[test]
fn start_of() {
// Genesis slot and genesis duration 0.

View File

@ -22,6 +22,11 @@ impl SlotClock for SystemTimeSlotClock {
self.clock.slot_of(now)
}
fn is_prior_to_genesis(&self) -> Option<bool> {
let now = SystemTime::now().duration_since(UNIX_EPOCH).ok()?;
Some(now < *self.clock.genesis_duration())
}
fn now_duration(&self) -> Option<Duration> {
SystemTime::now().duration_since(UNIX_EPOCH).ok()
}

View File

@ -0,0 +1,20 @@
[package]
name = "fork_choice"
version = "0.1.0"
authors = ["Paul Hauner <paul@paulhauner.com>"]
edition = "2018"
# See more keys and their definitions at https://doc.rust-lang.org/cargo/reference/manifest.html
[dependencies]
types = { path = "../types" }
proto_array = { path = "../proto_array" }
eth2_ssz = { path = "../ssz" }
eth2_ssz_derive = { path = "../ssz_derive" }
[dev-dependencies]
state_processing = { path = "../../consensus/state_processing" }
beacon_chain = { path = "../../beacon_node/beacon_chain" }
store = { path = "../../beacon_node/store" }
tree_hash = { path = "../../consensus/tree_hash" }
slot_clock = { path = "../../common/slot_clock" }

View File

@ -0,0 +1,884 @@
use crate::ForkChoiceStore;
use proto_array::{Block as ProtoBlock, ProtoArrayForkChoice};
use ssz_derive::{Decode, Encode};
use std::marker::PhantomData;
use types::{
BeaconBlock, BeaconState, BeaconStateError, Epoch, EthSpec, Hash256, IndexedAttestation, Slot,
};
/// Defined here:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#configuration
pub const SAFE_SLOTS_TO_UPDATE_JUSTIFIED: u64 = 8;
#[derive(Debug)]
pub enum Error<T> {
InvalidAttestation(InvalidAttestation),
InvalidBlock(InvalidBlock),
ProtoArrayError(String),
InvalidProtoArrayBytes(String),
MissingProtoArrayBlock(Hash256),
UnknownAncestor {
ancestor_slot: Slot,
descendant_root: Hash256,
},
InconsistentOnTick {
previous_slot: Slot,
time: Slot,
},
BeaconStateError(BeaconStateError),
AttemptToRevertJustification {
store: Slot,
state: Slot,
},
ForkChoiceStoreError(T),
UnableToSetJustifiedCheckpoint(T),
AfterBlockFailed(T),
}
impl<T> From<InvalidAttestation> for Error<T> {
fn from(e: InvalidAttestation) -> Self {
Error::InvalidAttestation(e)
}
}
#[derive(Debug)]
pub enum InvalidBlock {
UnknownParent(Hash256),
FutureSlot {
current_slot: Slot,
block_slot: Slot,
},
FinalizedSlot {
finalized_slot: Slot,
block_slot: Slot,
},
NotFinalizedDescendant {
finalized_root: Hash256,
block_ancestor: Option<Hash256>,
},
}
#[derive(Debug)]
pub enum InvalidAttestation {
/// The attestations aggregation bits were empty when they shouldn't be.
EmptyAggregationBitfield,
/// The `attestation.data.beacon_block_root` block is unknown.
UnknownHeadBlock { beacon_block_root: Hash256 },
/// The `attestation.data.slot` is not from the same epoch as `data.target.epoch` and therefore
/// the attestation is invalid.
BadTargetEpoch { target: Epoch, slot: Slot },
/// The target root of the attestation points to a block that we have not verified.
UnknownTargetRoot(Hash256),
/// The attestation is for an epoch in the future (with respect to the gossip clock disparity).
FutureEpoch {
attestation_epoch: Epoch,
current_epoch: Epoch,
},
/// The attestation is for an epoch in the past (with respect to the gossip clock disparity).
PastEpoch {
attestation_epoch: Epoch,
current_epoch: Epoch,
},
/// The attestation references a target root that does not match what is stored in our
/// database.
InvalidTarget {
attestation: Hash256,
local: Hash256,
},
/// The attestation is attesting to a state that is later than itself. (Viz., attesting to the
/// future).
AttestsToFutureBlock { block: Slot, attestation: Slot },
}
impl<T> From<String> for Error<T> {
fn from(e: String) -> Self {
Error::ProtoArrayError(e)
}
}
/// Calculate how far `slot` lies from the start of its epoch.
///
/// ## Specification
///
/// Equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#compute_slots_since_epoch_start
pub fn compute_slots_since_epoch_start<E: EthSpec>(slot: Slot) -> Slot {
slot - slot
.epoch(E::slots_per_epoch())
.start_slot(E::slots_per_epoch())
}
/// Calculate the first slot in `epoch`.
///
/// ## Specification
///
/// Equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/beacon-chain.md#compute_start_slot_at_epoch
fn compute_start_slot_at_epoch<E: EthSpec>(epoch: Epoch) -> Slot {
epoch.start_slot(E::slots_per_epoch())
}
/// Called whenever the current time increases.
///
/// ## Specification
///
/// Equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#on_tick
fn on_tick<T, E>(store: &mut T, time: Slot) -> Result<(), Error<T::Error>>
where
T: ForkChoiceStore<E>,
E: EthSpec,
{
let previous_slot = store.get_current_slot();
if time > previous_slot + 1 {
return Err(Error::InconsistentOnTick {
previous_slot,
time,
});
}
// Update store time.
store.set_current_slot(time);
let current_slot = store.get_current_slot();
if !(current_slot > previous_slot && compute_slots_since_epoch_start::<E>(current_slot) == 0) {
return Ok(());
}
if store.best_justified_checkpoint().epoch > store.justified_checkpoint().epoch {
store
.set_justified_checkpoint(*store.best_justified_checkpoint())
.map_err(Error::ForkChoiceStoreError)?;
}
Ok(())
}
/// Used for queuing attestations from the current slot. Only contains the minimum necessary
/// information about the attestation.
#[derive(Clone, PartialEq, Encode, Decode)]
pub struct QueuedAttestation {
slot: Slot,
attesting_indices: Vec<u64>,
block_root: Hash256,
target_epoch: Epoch,
}
impl<E: EthSpec> From<&IndexedAttestation<E>> for QueuedAttestation {
fn from(a: &IndexedAttestation<E>) -> Self {
Self {
slot: a.data.slot,
attesting_indices: a.attesting_indices[..].to_vec(),
block_root: a.data.beacon_block_root,
target_epoch: a.data.target.epoch,
}
}
}
/// Returns all values in `self.queued_attestations` that have a slot that is earlier than the
/// current slot. Also removes those values from `self.queued_attestations`.
fn dequeue_attestations(
current_slot: Slot,
queued_attestations: &mut Vec<QueuedAttestation>,
) -> Vec<QueuedAttestation> {
let remaining = queued_attestations.split_off(
queued_attestations
.iter()
.position(|a| a.slot >= current_slot)
.unwrap_or(queued_attestations.len()),
);
std::mem::replace(queued_attestations, remaining)
}
/// Provides an implementation of "Ethereum 2.0 Phase 0 -- Beacon Chain Fork Choice":
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#ethereum-20-phase-0----beacon-chain-fork-choice
///
/// ## Detail
///
/// This struct wraps `ProtoArrayForkChoice` and provides:
///
/// - Management of the justified state and caching of balances.
/// - Queuing of attestations from the current slot.
pub struct ForkChoice<T, E> {
/// Storage for `ForkChoice`, modelled off the spec `Store` object.
fc_store: T,
/// The underlying representation of the block DAG.
proto_array: ProtoArrayForkChoice,
/// Attestations that arrived at the current slot and must be queued for later processing.
queued_attestations: Vec<QueuedAttestation>,
_phantom: PhantomData<E>,
}
impl<T, E> PartialEq for ForkChoice<T, E>
where
T: ForkChoiceStore<E> + PartialEq,
E: EthSpec,
{
fn eq(&self, other: &Self) -> bool {
self.fc_store == other.fc_store
&& self.proto_array == other.proto_array
&& self.queued_attestations == other.queued_attestations
}
}
impl<T, E> ForkChoice<T, E>
where
T: ForkChoiceStore<E>,
E: EthSpec,
{
/// Instantiates `Self` from the genesis parameters.
pub fn from_genesis(
fc_store: T,
genesis_block: &BeaconBlock<E>,
) -> Result<Self, Error<T::Error>> {
let finalized_block_slot = genesis_block.slot;
let finalized_block_state_root = genesis_block.state_root;
let proto_array = ProtoArrayForkChoice::new(
finalized_block_slot,
finalized_block_state_root,
fc_store.justified_checkpoint().epoch,
fc_store.finalized_checkpoint().epoch,
fc_store.finalized_checkpoint().root,
)?;
Ok(Self {
fc_store,
proto_array,
queued_attestations: vec![],
_phantom: PhantomData,
})
}
/// Instantiates `Self` from some existing components.
///
/// This is useful if the existing components have been loaded from disk after a process
/// restart.
pub fn from_components(
fc_store: T,
proto_array: ProtoArrayForkChoice,
queued_attestations: Vec<QueuedAttestation>,
) -> Self {
Self {
fc_store,
proto_array,
queued_attestations,
_phantom: PhantomData,
}
}
/// Returns the block root of an ancestor of `block_root` at the given `slot`. (Note: `slot` refers
/// to the block that is *returned*, not the one that is supplied.)
///
/// The result may be `Ok(None)` if the block does not descend from the finalized block. This
/// is an artifact of proto-array, sometimes it contains descendants of blocks that have been
/// pruned.
///
/// ## Specification
///
/// Equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#get_ancestor
fn get_ancestor(
&self,
block_root: Hash256,
ancestor_slot: Slot,
) -> Result<Option<Hash256>, Error<T::Error>>
where
T: ForkChoiceStore<E>,
E: EthSpec,
{
let block = self
.proto_array
.get_block(&block_root)
.ok_or_else(|| Error::MissingProtoArrayBlock(block_root))?;
if block.slot > ancestor_slot {
Ok(self
.proto_array
.core_proto_array()
.iter_block_roots(&block_root)
// Search for a slot that is **less than or equal to** the target slot. We check
// for lower slots to account for skip slots.
.find(|(_, slot)| *slot <= ancestor_slot)
.map(|(root, _)| root))
} else if block.slot == ancestor_slot {
Ok(Some(block_root))
} else {
// Root is older than queried slot, thus a skip slot. Return most recent root prior to
// slot.
Ok(Some(block_root))
}
}
/// Run the fork choice rule to determine the head.
///
/// ## Specification
///
/// Is equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#get_head
pub fn get_head(&mut self, current_slot: Slot) -> Result<Hash256, Error<T::Error>> {
self.update_time(current_slot)?;
let store = &mut self.fc_store;
let result = self
.proto_array
.find_head(
store.justified_checkpoint().epoch,
store.justified_checkpoint().root,
store.finalized_checkpoint().epoch,
store.justified_balances(),
)
.map_err(Into::into);
result
}
/// Returns `true` if the given `store` should be updated to set
/// `state.current_justified_checkpoint` its `justified_checkpoint`.
///
/// ## Specification
///
/// Is equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#should_update_justified_checkpoint
fn should_update_justified_checkpoint(
&mut self,
current_slot: Slot,
state: &BeaconState<E>,
) -> Result<bool, Error<T::Error>> {
self.update_time(current_slot)?;
let new_justified_checkpoint = &state.current_justified_checkpoint;
if compute_slots_since_epoch_start::<E>(self.fc_store.get_current_slot())
< SAFE_SLOTS_TO_UPDATE_JUSTIFIED
{
return Ok(true);
}
let justified_slot =
compute_start_slot_at_epoch::<E>(self.fc_store.justified_checkpoint().epoch);
// This sanity check is not in the spec, but the invariant is implied.
if justified_slot >= state.slot {
return Err(Error::AttemptToRevertJustification {
store: justified_slot,
state: state.slot,
});
}
// We know that the slot for `new_justified_checkpoint.root` is not greater than
// `state.slot`, since a state cannot justify its own slot.
//
// We know that `new_justified_checkpoint.root` is an ancestor of `state`, since a `state`
// only ever justifies ancestors.
//
// A prior `if` statement protects against a justified_slot that is greater than
// `state.slot`
let justified_ancestor =
self.get_ancestor(new_justified_checkpoint.root, justified_slot)?;
if justified_ancestor != Some(self.fc_store.justified_checkpoint().root) {
return Ok(false);
}
Ok(true)
}
/// Add `block` to the fork choice DAG.
///
/// - `block_root` is the root of `block.
/// - The root of `state` matches `block.state_root`.
///
/// ## Specification
///
/// Approximates:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#on_block
///
/// It only approximates the specification since it does not run the `state_transition` check.
/// That should have already been called upstream and it's too expensive to call again.
///
/// ## Notes:
///
/// The supplied block **must** pass the `state_transition` function as it will not be run
/// here.
pub fn on_block(
&mut self,
current_slot: Slot,
block: &BeaconBlock<E>,
block_root: Hash256,
state: &BeaconState<E>,
) -> Result<(), Error<T::Error>> {
let current_slot = self.update_time(current_slot)?;
// Parent block must be known.
if !self.proto_array.contains_block(&block.parent_root) {
return Err(Error::InvalidBlock(InvalidBlock::UnknownParent(
block.parent_root,
)));
}
// Blocks cannot be in the future. If they are, their consideration must be delayed until
// the are in the past.
//
// Note: presently, we do not delay consideration. We just drop the block.
if block.slot > current_slot {
return Err(Error::InvalidBlock(InvalidBlock::FutureSlot {
current_slot,
block_slot: block.slot,
}));
}
// Check that block is later than the finalized epoch slot (optimization to reduce calls to
// get_ancestor).
let finalized_slot =
compute_start_slot_at_epoch::<E>(self.fc_store.finalized_checkpoint().epoch);
if block.slot <= finalized_slot {
return Err(Error::InvalidBlock(InvalidBlock::FinalizedSlot {
finalized_slot,
block_slot: block.slot,
}));
}
// Check block is a descendant of the finalized block at the checkpoint finalized slot.
//
// Note: the specification uses `hash_tree_root(block)` instead of `block.parent_root` for
// the start of this search. I claim that since `block.slot > finalized_slot` it is
// equivalent to use the parent root for this search. Doing so reduces a single lookup
// (trivial), but more importantly, it means we don't need to have added `block` to
// `self.proto_array` to do this search. See:
//
// https://github.com/ethereum/eth2.0-specs/pull/1884
let block_ancestor = self.get_ancestor(block.parent_root, finalized_slot)?;
let finalized_root = self.fc_store.finalized_checkpoint().root;
if block_ancestor != Some(finalized_root) {
return Err(Error::InvalidBlock(InvalidBlock::NotFinalizedDescendant {
finalized_root,
block_ancestor,
}));
}
// Update justified checkpoint.
if state.current_justified_checkpoint.epoch > self.fc_store.justified_checkpoint().epoch {
if state.current_justified_checkpoint.epoch
> self.fc_store.best_justified_checkpoint().epoch
{
self.fc_store
.set_best_justified_checkpoint(state.current_justified_checkpoint);
}
if self.should_update_justified_checkpoint(current_slot, state)? {
self.fc_store
.set_justified_checkpoint(state.current_justified_checkpoint)
.map_err(Error::UnableToSetJustifiedCheckpoint)?;
}
}
// Update finalized checkpoint.
if state.finalized_checkpoint.epoch > self.fc_store.finalized_checkpoint().epoch {
self.fc_store
.set_finalized_checkpoint(state.finalized_checkpoint);
let finalized_slot =
compute_start_slot_at_epoch::<E>(self.fc_store.finalized_checkpoint().epoch);
// Note: the `if` statement here is not part of the specification, but I claim that it
// is an optimization and equivalent to the specification. See this PR for more
// information:
//
// https://github.com/ethereum/eth2.0-specs/pull/1880
if *self.fc_store.justified_checkpoint() != state.current_justified_checkpoint {
if state.current_justified_checkpoint.epoch
> self.fc_store.justified_checkpoint().epoch
|| self
.get_ancestor(self.fc_store.justified_checkpoint().root, finalized_slot)?
!= Some(self.fc_store.finalized_checkpoint().root)
{
self.fc_store
.set_justified_checkpoint(state.current_justified_checkpoint)
.map_err(Error::UnableToSetJustifiedCheckpoint)?;
}
}
}
let target_slot = block
.slot
.epoch(E::slots_per_epoch())
.start_slot(E::slots_per_epoch());
let target_root = if block.slot == target_slot {
block_root
} else {
*state
.get_block_root(target_slot)
.map_err(Error::BeaconStateError)?
};
self.fc_store
.on_verified_block(block, block_root, state)
.map_err(Error::AfterBlockFailed)?;
// This does not apply a vote to the block, it just makes fork choice aware of the block so
// it can still be identified as the head even if it doesn't have any votes.
self.proto_array.process_block(ProtoBlock {
slot: block.slot,
root: block_root,
parent_root: Some(block.parent_root),
target_root,
state_root: block.state_root,
justified_epoch: state.current_justified_checkpoint.epoch,
finalized_epoch: state.finalized_checkpoint.epoch,
})?;
Ok(())
}
/// Validates the `indexed_attestation` for application to fork choice.
///
/// ## Specification
///
/// Equivalent to:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.1/specs/phase0/fork-choice.md#validate_on_attestation
fn validate_on_attestation(
&self,
indexed_attestation: &IndexedAttestation<E>,
) -> Result<(), InvalidAttestation> {
// There is no point in processing an attestation with an empty bitfield. Reject
// it immediately.
//
// This is not in the specification, however it should be transparent to other nodes. We
// return early here to avoid wasting precious resources verifying the rest of it.
if indexed_attestation.attesting_indices.len() == 0 {
return Err(InvalidAttestation::EmptyAggregationBitfield);
}
let slot_now = self.fc_store.get_current_slot();
let epoch_now = slot_now.epoch(E::slots_per_epoch());
let target = indexed_attestation.data.target.clone();
// Attestation must be from the current or previous epoch.
if target.epoch > epoch_now {
return Err(InvalidAttestation::FutureEpoch {
attestation_epoch: target.epoch,
current_epoch: epoch_now,
});
} else if target.epoch + 1 < epoch_now {
return Err(InvalidAttestation::PastEpoch {
attestation_epoch: target.epoch,
current_epoch: epoch_now,
});
}
if target.epoch != indexed_attestation.data.slot.epoch(E::slots_per_epoch()) {
return Err(InvalidAttestation::BadTargetEpoch {
target: target.epoch,
slot: indexed_attestation.data.slot,
});
}
// Attestation target must be for a known block.
//
// We do not delay the block for later processing to reduce complexity and DoS attack
// surface.
if !self.proto_array.contains_block(&target.root) {
return Err(InvalidAttestation::UnknownTargetRoot(target.root));
}
// Load the block for `attestation.data.beacon_block_root`.
//
// This indirectly checks to see if the `attestation.data.beacon_block_root` is in our fork
// choice. Any known, non-finalized block should be in fork choice, so this check
// immediately filters out attestations that attest to a block that has not been processed.
//
// Attestations must be for a known block. If the block is unknown, we simply drop the
// attestation and do not delay consideration for later.
let block = self
.proto_array
.get_block(&indexed_attestation.data.beacon_block_root)
.ok_or_else(|| InvalidAttestation::UnknownHeadBlock {
beacon_block_root: indexed_attestation.data.beacon_block_root,
})?;
if block.target_root != target.root {
return Err(InvalidAttestation::InvalidTarget {
attestation: target.root,
local: block.target_root,
});
}
// Attestations must not be for blocks in the future. If this is the case, the attestation
// should not be considered.
if block.slot > indexed_attestation.data.slot {
return Err(InvalidAttestation::AttestsToFutureBlock {
block: block.slot,
attestation: indexed_attestation.data.slot,
});
}
Ok(())
}
/// Register `attestation` with the fork choice DAG so that it may influence future calls to
/// `Self::get_head`.
///
/// ## Specification
///
/// Approximates:
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#on_attestation
///
/// It only approximates the specification since it does not perform
/// `is_valid_indexed_attestation` since that should already have been called upstream and it's
/// too expensive to call again.
///
/// ## Notes:
///
/// The supplied `attestation` **must** pass the `in_valid_indexed_attestation` function as it
/// will not be run here.
pub fn on_attestation(
&mut self,
current_slot: Slot,
attestation: &IndexedAttestation<E>,
) -> Result<(), Error<T::Error>> {
// Ensure the store is up-to-date.
self.update_time(current_slot)?;
// Ignore any attestations to the zero hash.
//
// This is an edge case that results from the spec aliasing the zero hash to the genesis
// block. Attesters may attest to the zero hash if they have never seen a block.
//
// We have two options here:
//
// 1. Apply all zero-hash attestations to the genesis block.
// 2. Ignore all attestations to the zero hash.
//
// (1) becomes weird once we hit finality and fork choice drops the genesis block. (2) is
// fine because votes to the genesis block are not useful; all validators implicitly attest
// to genesis just by being present in the chain.
if attestation.data.beacon_block_root == Hash256::zero() {
return Ok(());
}
self.validate_on_attestation(attestation)?;
if attestation.data.slot < self.fc_store.get_current_slot() {
for validator_index in attestation.attesting_indices.iter() {
self.proto_array.process_attestation(
*validator_index as usize,
attestation.data.beacon_block_root,
attestation.data.target.epoch,
)?;
}
} else {
// The spec declares:
//
// ```
// Attestations can only affect the fork choice of subsequent slots.
// Delay consideration in the fork choice until their slot is in the past.
// ```
self.queued_attestations
.push(QueuedAttestation::from(attestation));
}
Ok(())
}
/// Call `on_tick` for all slots between `fc_store.get_current_slot()` and the provided
/// `current_slot`. Returns the value of `self.fc_store.get_current_slot`.
pub fn update_time(&mut self, current_slot: Slot) -> Result<Slot, Error<T::Error>> {
while self.fc_store.get_current_slot() < current_slot {
let previous_slot = self.fc_store.get_current_slot();
// Note: we are relying upon `on_tick` to update `fc_store.time` to ensure we don't
// get stuck in a loop.
on_tick(&mut self.fc_store, previous_slot + 1)?
}
// Process any attestations that might now be eligible.
self.process_attestation_queue()?;
Ok(self.fc_store.get_current_slot())
}
/// Processes and removes from the queue any queued attestations which may now be eligible for
/// processing due to the slot clock incrementing.
fn process_attestation_queue(&mut self) -> Result<(), Error<T::Error>> {
for attestation in dequeue_attestations(
self.fc_store.get_current_slot(),
&mut self.queued_attestations,
) {
for validator_index in attestation.attesting_indices.iter() {
self.proto_array.process_attestation(
*validator_index as usize,
attestation.block_root,
attestation.target_epoch,
)?;
}
}
Ok(())
}
/// Returns `true` if the block is known.
pub fn contains_block(&self, block_root: &Hash256) -> bool {
self.proto_array.contains_block(block_root)
}
/// Returns a `ProtoBlock` if the block is known.
pub fn get_block(&self, block_root: &Hash256) -> Option<ProtoBlock> {
self.proto_array.get_block(block_root)
}
/// Returns the latest message for a given validator, if any.
///
/// Returns `(block_root, block_slot)`.
///
/// ## Notes
///
/// It may be prudent to call `Self::update_time` before calling this function,
/// since some attestations might be queued and awaiting processing.
pub fn latest_message(&self, validator_index: usize) -> Option<(Hash256, Epoch)> {
self.proto_array.latest_message(validator_index)
}
/// Returns a reference to the underlying fork choice DAG.
pub fn proto_array(&self) -> &ProtoArrayForkChoice {
&self.proto_array
}
/// Returns a reference to the underlying `fc_store`.
pub fn fc_store(&self) -> &T {
&self.fc_store
}
/// Returns a reference to the currently queued attestations.
pub fn queued_attestations(&self) -> &[QueuedAttestation] {
&self.queued_attestations
}
/// Prunes the underlying fork choice DAG.
pub fn prune(&mut self) -> Result<(), Error<T::Error>> {
let finalized_root = self.fc_store.finalized_checkpoint().root;
self.proto_array
.maybe_prune(finalized_root)
.map_err(Into::into)
}
/// Instantiate `Self` from some `PersistedForkChoice` generated by a earlier call to
/// `Self::to_persisted`.
pub fn from_persisted(
persisted: PersistedForkChoice,
fc_store: T,
) -> Result<Self, Error<T::Error>> {
let proto_array = ProtoArrayForkChoice::from_bytes(&persisted.proto_array_bytes)
.map_err(Error::InvalidProtoArrayBytes)?;
Ok(Self {
fc_store,
proto_array,
queued_attestations: persisted.queued_attestations,
_phantom: PhantomData,
})
}
/// Takes a snapshot of `Self` and stores it in `PersistedForkChoice`, allowing this struct to
/// be instantiated again later.
pub fn to_persisted(&self) -> PersistedForkChoice {
PersistedForkChoice {
proto_array_bytes: self.proto_array().as_bytes(),
queued_attestations: self.queued_attestations().to_vec(),
}
}
}
/// Helper struct that is used to encode/decode the state of the `ForkChoice` as SSZ bytes.
///
/// This is used when persisting the state of the fork choice to disk.
#[derive(Encode, Decode, Clone)]
pub struct PersistedForkChoice {
proto_array_bytes: Vec<u8>,
queued_attestations: Vec<QueuedAttestation>,
}
#[cfg(test)]
mod tests {
use super::*;
use types::{EthSpec, MainnetEthSpec};
type E = MainnetEthSpec;
#[test]
fn slots_since_epoch_start() {
for epoch in 0..3 {
for slot in 0..E::slots_per_epoch() {
let input = epoch * E::slots_per_epoch() + slot;
assert_eq!(compute_slots_since_epoch_start::<E>(Slot::new(input)), slot)
}
}
}
#[test]
fn start_slot_at_epoch() {
for epoch in 0..3 {
assert_eq!(
compute_start_slot_at_epoch::<E>(Epoch::new(epoch)),
epoch * E::slots_per_epoch()
)
}
}
fn get_queued_attestations() -> Vec<QueuedAttestation> {
(1..4)
.into_iter()
.map(|i| QueuedAttestation {
slot: Slot::new(i),
attesting_indices: vec![],
block_root: Hash256::zero(),
target_epoch: Epoch::new(0),
})
.collect()
}
fn get_slots(queued_attestations: &[QueuedAttestation]) -> Vec<u64> {
queued_attestations.iter().map(|a| a.slot.into()).collect()
}
fn test_queued_attestations(current_time: Slot) -> (Vec<u64>, Vec<u64>) {
let mut queued = get_queued_attestations();
let dequeued = dequeue_attestations(current_time, &mut queued);
(get_slots(&queued), get_slots(&dequeued))
}
#[test]
fn dequeing_attestations() {
let (queued, dequeued) = test_queued_attestations(Slot::new(0));
assert_eq!(queued, vec![1, 2, 3]);
assert!(dequeued.is_empty());
let (queued, dequeued) = test_queued_attestations(Slot::new(1));
assert_eq!(queued, vec![1, 2, 3]);
assert!(dequeued.is_empty());
let (queued, dequeued) = test_queued_attestations(Slot::new(2));
assert_eq!(queued, vec![2, 3]);
assert_eq!(dequeued, vec![1]);
let (queued, dequeued) = test_queued_attestations(Slot::new(3));
assert_eq!(queued, vec![3]);
assert_eq!(dequeued, vec![1, 2]);
let (queued, dequeued) = test_queued_attestations(Slot::new(4));
assert!(queued.is_empty());
assert_eq!(dequeued, vec![1, 2, 3]);
}
}

View File

@ -0,0 +1,61 @@
use types::{BeaconBlock, BeaconState, Checkpoint, EthSpec, Hash256, Slot};
/// Approximates the `Store` in "Ethereum 2.0 Phase 0 -- Beacon Chain Fork Choice":
///
/// https://github.com/ethereum/eth2.0-specs/blob/v0.12.0/specs/phase0/fork-choice.md#store
///
/// ## Detail
///
/// This is only an approximation for two reasons:
///
/// - This crate stores the actual block DAG in `ProtoArrayForkChoice`.
/// - `time` is represented using `Slot` instead of UNIX epoch `u64`.
///
/// ## Motiviation
///
/// The primary motivation for defining this as a trait to be implemented upstream rather than a
/// concrete struct is to allow this crate to be free from "impure" on-disk database logic,
/// hopefully making auditing easier.
pub trait ForkChoiceStore<T: EthSpec>: Sized {
type Error;
/// Returns the last value passed to `Self::update_time`.
fn get_current_slot(&self) -> Slot;
/// Set the value to be returned by `Self::get_current_slot`.
///
/// ## Notes
///
/// This should only ever be called from within `ForkChoice::on_tick`.
fn set_current_slot(&mut self, slot: Slot);
/// Called whenever `ForkChoice::on_block` has verified a block, but not yet added it to fork
/// choice. Allows the implementer to performing caching or other housekeeping duties.
fn on_verified_block(
&mut self,
block: &BeaconBlock<T>,
block_root: Hash256,
state: &BeaconState<T>,
) -> Result<(), Self::Error>;
/// Returns the `justified_checkpoint`.
fn justified_checkpoint(&self) -> &Checkpoint;
/// Returns balances from the `state` identified by `justified_checkpoint.root`.
fn justified_balances(&self) -> &[u64];
/// Returns the `best_justified_checkpoint`.
fn best_justified_checkpoint(&self) -> &Checkpoint;
/// Returns the `finalized_checkpoint`.
fn finalized_checkpoint(&self) -> &Checkpoint;
/// Sets `finalized_checkpoint`.
fn set_finalized_checkpoint(&mut self, checkpoint: Checkpoint);
/// Sets the `justified_checkpoint`.
fn set_justified_checkpoint(&mut self, checkpoint: Checkpoint) -> Result<(), Self::Error>;
/// Sets the `best_justified_checkpoint`.
fn set_best_justified_checkpoint(&mut self, checkpoint: Checkpoint);
}

View File

@ -0,0 +1,8 @@
mod fork_choice;
mod fork_choice_store;
pub use crate::fork_choice::{
Error, ForkChoice, InvalidAttestation, InvalidBlock, PersistedForkChoice, QueuedAttestation,
SAFE_SLOTS_TO_UPDATE_JUSTIFIED,
};
pub use fork_choice_store::ForkChoiceStore;

View File

@ -0,0 +1,802 @@
#![cfg(not(debug_assertions))]
use beacon_chain::{
test_utils::{AttestationStrategy, BeaconChainHarness, BlockStrategy, HarnessType},
BeaconChain, BeaconChainError, BeaconForkChoiceStore, ForkChoiceError,
};
use fork_choice::{
ForkChoiceStore, InvalidAttestation, InvalidBlock, QueuedAttestation,
SAFE_SLOTS_TO_UPDATE_JUSTIFIED,
};
use std::sync::Mutex;
use store::{MemoryStore, Store};
use types::{
test_utils::{generate_deterministic_keypair, generate_deterministic_keypairs},
Epoch, EthSpec, IndexedAttestation, MainnetEthSpec, Slot,
};
use types::{BeaconBlock, BeaconState, Hash256, SignedBeaconBlock};
pub type E = MainnetEthSpec;
pub const VALIDATOR_COUNT: usize = 16;
/// Defines some delay between when an attestation is created and when it is mutated.
pub enum MutationDelay {
/// No delay between creation and mutation.
NoDelay,
/// Create `n` blocks before mutating the attestation.
Blocks(usize),
}
/// A helper struct to make testing fork choice more ergonomic and less repetitive.
struct ForkChoiceTest {
harness: BeaconChainHarness<HarnessType<E>>,
}
impl ForkChoiceTest {
/// Creates a new tester.
pub fn new() -> Self {
let harness = BeaconChainHarness::new_with_target_aggregators(
MainnetEthSpec,
generate_deterministic_keypairs(VALIDATOR_COUNT),
// Ensure we always have an aggregator for each slot.
u64::max_value(),
);
Self { harness }
}
/// Get a value from the `ForkChoice` instantiation.
fn get<T, U>(&self, func: T) -> U
where
T: Fn(&BeaconForkChoiceStore<MemoryStore<E>, E>) -> U,
{
func(&self.harness.chain.fork_choice.read().fc_store())
}
/// Assert the epochs match.
pub fn assert_finalized_epoch(self, epoch: u64) -> Self {
assert_eq!(
self.get(|fc_store| fc_store.finalized_checkpoint().epoch),
Epoch::new(epoch),
"finalized_epoch"
);
self
}
/// Assert the epochs match.
pub fn assert_justified_epoch(self, epoch: u64) -> Self {
assert_eq!(
self.get(|fc_store| fc_store.justified_checkpoint().epoch),
Epoch::new(epoch),
"justified_epoch"
);
self
}
/// Assert the epochs match.
pub fn assert_best_justified_epoch(self, epoch: u64) -> Self {
assert_eq!(
self.get(|fc_store| fc_store.best_justified_checkpoint().epoch),
Epoch::new(epoch),
"best_justified_epoch"
);
self
}
/// Inspect the queued attestations in fork choice.
pub fn inspect_queued_attestations<F>(self, mut func: F) -> Self
where
F: FnMut(&[QueuedAttestation]),
{
self.harness
.chain
.fork_choice
.write()
.update_time(self.harness.chain.slot().unwrap())
.unwrap();
func(self.harness.chain.fork_choice.read().queued_attestations());
self
}
/// Skip a slot, without producing a block.
pub fn skip_slot(self) -> Self {
self.harness.advance_slot();
self
}
/// Build the chain whilst `predicate` returns `true`.
pub fn apply_blocks_while<F>(self, mut predicate: F) -> Self
where
F: FnMut(&BeaconBlock<E>, &BeaconState<E>) -> bool,
{
self.harness.advance_slot();
self.harness.extend_chain_while(
|block, state| predicate(&block.message, state),
BlockStrategy::OnCanonicalHead,
AttestationStrategy::AllValidators,
);
self
}
/// Apply `count` blocks to the chain (with attestations).
pub fn apply_blocks(self, count: usize) -> Self {
self.harness.advance_slot();
self.harness.extend_chain(
count,
BlockStrategy::OnCanonicalHead,
AttestationStrategy::AllValidators,
);
self
}
/// Apply `count` blocks to the chain (without attestations).
pub fn apply_blocks_without_new_attestations(self, count: usize) -> Self {
self.harness.advance_slot();
self.harness.extend_chain(
count,
BlockStrategy::OnCanonicalHead,
AttestationStrategy::SomeValidators(vec![]),
);
self
}
/// Moves to the next slot that is *outside* the `SAFE_SLOTS_TO_UPDATE_JUSTIFIED` range.
///
/// If the chain is presently in an unsafe period, transition through it and the following safe
/// period.
pub fn move_to_next_unsafe_period(self) -> Self {
self.move_inside_safe_to_update()
.move_outside_safe_to_update()
}
/// Moves to the next slot that is *outside* the `SAFE_SLOTS_TO_UPDATE_JUSTIFIED` range.
pub fn move_outside_safe_to_update(self) -> Self {
while is_safe_to_update(self.harness.chain.slot().unwrap()) {
self.harness.advance_slot()
}
self
}
/// Moves to the next slot that is *inside* the `SAFE_SLOTS_TO_UPDATE_JUSTIFIED` range.
pub fn move_inside_safe_to_update(self) -> Self {
while !is_safe_to_update(self.harness.chain.slot().unwrap()) {
self.harness.advance_slot()
}
self
}
/// Applies a block directly to fork choice, bypassing the beacon chain.
///
/// Asserts the block was applied successfully.
pub fn apply_block_directly_to_fork_choice<F>(self, mut func: F) -> Self
where
F: FnMut(&mut BeaconBlock<E>, &mut BeaconState<E>),
{
let (mut block, mut state) = self.harness.get_block();
func(&mut block.message, &mut state);
let current_slot = self.harness.chain.slot().unwrap();
self.harness
.chain
.fork_choice
.write()
.on_block(current_slot, &block.message, block.canonical_root(), &state)
.unwrap();
self
}
/// Applies a block directly to fork choice, bypassing the beacon chain.
///
/// Asserts that an error occurred and allows inspecting it via `comparison_func`.
pub fn apply_invalid_block_directly_to_fork_choice<F, G>(
self,
mut mutation_func: F,
mut comparison_func: G,
) -> Self
where
F: FnMut(&mut BeaconBlock<E>, &mut BeaconState<E>),
G: FnMut(ForkChoiceError),
{
let (mut block, mut state) = self.harness.get_block();
mutation_func(&mut block.message, &mut state);
let current_slot = self.harness.chain.slot().unwrap();
let err = self
.harness
.chain
.fork_choice
.write()
.on_block(current_slot, &block.message, block.canonical_root(), &state)
.err()
.expect("on_block did not return an error");
comparison_func(err);
self
}
/// Compares the justified balances in the `ForkChoiceStore` verses a direct lookup from the
/// database.
fn check_justified_balances(&self) {
let harness = &self.harness;
let fc = self.harness.chain.fork_choice.read();
let state_root = harness
.chain
.store
.get_item::<SignedBeaconBlock<E>>(&fc.fc_store().justified_checkpoint().root)
.unwrap()
.unwrap()
.message
.state_root;
let state = harness
.chain
.store
.get_state(&state_root, None)
.unwrap()
.unwrap();
let balances = state
.validators
.into_iter()
.map(|v| {
if v.is_active_at(state.current_epoch()) {
v.effective_balance
} else {
0
}
})
.collect::<Vec<_>>();
assert_eq!(
&balances[..],
fc.fc_store().justified_balances(),
"balances should match"
)
}
/// Returns an attestation that is valid for some slot in the given `chain`.
///
/// Also returns some info about who created it.
fn apply_attestation_to_chain<F, G>(
self,
delay: MutationDelay,
mut mutation_func: F,
mut comparison_func: G,
) -> Self
where
F: FnMut(&mut IndexedAttestation<E>, &BeaconChain<HarnessType<E>>),
G: FnMut(Result<(), BeaconChainError>),
{
let chain = &self.harness.chain;
let head = chain.head().expect("should get head");
let current_slot = chain.slot().expect("should get slot");
let mut attestation = chain
.produce_unaggregated_attestation(current_slot, 0)
.expect("should not error while producing attestation");
let validator_committee_index = 0;
let validator_index = *head
.beacon_state
.get_beacon_committee(current_slot, attestation.data.index)
.expect("should get committees")
.committee
.get(validator_committee_index)
.expect("there should be an attesting validator");
let validator_sk = generate_deterministic_keypair(validator_index).sk;
attestation
.sign(
&validator_sk,
validator_committee_index,
&head.beacon_state.fork,
chain.genesis_validators_root,
&chain.spec,
)
.expect("should sign attestation");
let mut verified_attestation = chain
.verify_unaggregated_attestation_for_gossip(attestation)
.expect("precondition: should gossip verify attestation");
if let MutationDelay::Blocks(slots) = delay {
self.harness.advance_slot();
self.harness.extend_chain(
slots,
BlockStrategy::OnCanonicalHead,
AttestationStrategy::SomeValidators(vec![]),
);
}
mutation_func(verified_attestation.__indexed_attestation_mut(), chain);
let result = chain.apply_attestation_to_fork_choice(&verified_attestation);
comparison_func(result);
self
}
}
fn is_safe_to_update(slot: Slot) -> bool {
slot % E::slots_per_epoch() < SAFE_SLOTS_TO_UPDATE_JUSTIFIED
}
/// - The new justified checkpoint descends from the current.
/// - Current slot is within `SAFE_SLOTS_TO_UPDATE_JUSTIFIED`
#[test]
fn justified_checkpoint_updates_with_descendent_inside_safe_slots() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch == 0)
.move_inside_safe_to_update()
.assert_justified_epoch(0)
.apply_blocks(1)
.assert_justified_epoch(2);
}
/// - The new justified checkpoint descends from the current.
/// - Current slot is **not** within `SAFE_SLOTS_TO_UPDATE_JUSTIFIED`
/// - This is **not** the first justification since genesis
#[test]
fn justified_checkpoint_updates_with_descendent_outside_safe_slots() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch <= 2)
.move_outside_safe_to_update()
.assert_justified_epoch(2)
.assert_best_justified_epoch(2)
.apply_blocks(1)
.assert_justified_epoch(3);
}
/// - The new justified checkpoint descends from the current.
/// - Current slot is **not** within `SAFE_SLOTS_TO_UPDATE_JUSTIFIED`
/// - This is the first justification since genesis
#[test]
fn justified_checkpoint_updates_first_justification_outside_safe_to_update() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch == 0)
.move_to_next_unsafe_period()
.assert_justified_epoch(0)
.assert_best_justified_epoch(0)
.apply_blocks(1)
.assert_justified_epoch(2)
.assert_best_justified_epoch(2);
}
/// - The new justified checkpoint **does not** descend from the current.
/// - Current slot is within `SAFE_SLOTS_TO_UPDATE_JUSTIFIED`
/// - Finalized epoch has **not** increased.
#[test]
fn justified_checkpoint_updates_with_non_descendent_inside_safe_slots_without_finality() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch == 0)
.apply_blocks(1)
.move_inside_safe_to_update()
.assert_justified_epoch(2)
.apply_block_directly_to_fork_choice(|_, state| {
// The finalized checkpoint should not change.
state.finalized_checkpoint.epoch = Epoch::new(0);
// The justified checkpoint has changed.
state.current_justified_checkpoint.epoch = Epoch::new(3);
// The new block should **not** include the current justified block as an ancestor.
state.current_justified_checkpoint.root = *state
.get_block_root(Epoch::new(1).start_slot(E::slots_per_epoch()))
.unwrap();
})
.assert_justified_epoch(3)
.assert_best_justified_epoch(3);
}
/// - The new justified checkpoint **does not** descend from the current.
/// - Current slot is **not** within `SAFE_SLOTS_TO_UPDATE_JUSTIFIED`.
/// - Finalized epoch has **not** increased.
#[test]
fn justified_checkpoint_updates_with_non_descendent_outside_safe_slots_without_finality() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch == 0)
.apply_blocks(1)
.move_to_next_unsafe_period()
.assert_justified_epoch(2)
.apply_block_directly_to_fork_choice(|_, state| {
// The finalized checkpoint should not change.
state.finalized_checkpoint.epoch = Epoch::new(0);
// The justified checkpoint has changed.
state.current_justified_checkpoint.epoch = Epoch::new(3);
// The new block should **not** include the current justified block as an ancestor.
state.current_justified_checkpoint.root = *state
.get_block_root(Epoch::new(1).start_slot(E::slots_per_epoch()))
.unwrap();
})
.assert_justified_epoch(2)
.assert_best_justified_epoch(3);
}
/// - The new justified checkpoint **does not** descend from the current.
/// - Current slot is **not** within `SAFE_SLOTS_TO_UPDATE_JUSTIFIED`
/// - Finalized epoch has increased.
#[test]
fn justified_checkpoint_updates_with_non_descendent_outside_safe_slots_with_finality() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch == 0)
.apply_blocks(1)
.move_to_next_unsafe_period()
.assert_justified_epoch(2)
.apply_block_directly_to_fork_choice(|_, state| {
// The finalized checkpoint should change.
state.finalized_checkpoint.epoch = Epoch::new(1);
// The justified checkpoint has changed.
state.current_justified_checkpoint.epoch = Epoch::new(3);
// The new block should **not** include the current justified block as an ancestor.
state.current_justified_checkpoint.root = *state
.get_block_root(Epoch::new(1).start_slot(E::slots_per_epoch()))
.unwrap();
})
.assert_justified_epoch(3)
.assert_best_justified_epoch(3);
}
/// Check that the balances are obtained correctly.
#[test]
fn justified_balances() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.current_justified_checkpoint.epoch == 0)
.apply_blocks(1)
.assert_justified_epoch(2)
.check_justified_balances()
}
macro_rules! assert_invalid_block {
($err: tt, $($error: pat) |+ $( if $guard: expr )?) => {
assert!(
matches!(
$err,
$( ForkChoiceError::InvalidBlock($error) ) |+ $( if $guard )?
),
);
};
}
/// Specification v0.12.1
///
/// assert block.parent_root in store.block_states
#[test]
fn invalid_block_unknown_parent() {
let junk = Hash256::from_low_u64_be(42);
ForkChoiceTest::new()
.apply_blocks(2)
.apply_invalid_block_directly_to_fork_choice(
|block, _| {
block.parent_root = junk;
},
|err| {
assert_invalid_block!(
err,
InvalidBlock::UnknownParent(parent)
if parent == junk
)
},
);
}
/// Specification v0.12.1
///
/// assert get_current_slot(store) >= block.slot
#[test]
fn invalid_block_future_slot() {
ForkChoiceTest::new()
.apply_blocks(2)
.apply_invalid_block_directly_to_fork_choice(
|block, _| {
block.slot = block.slot + 1;
},
|err| {
assert_invalid_block!(
err,
InvalidBlock::FutureSlot { .. }
)
},
);
}
/// Specification v0.12.1
///
/// assert block.slot > finalized_slot
#[test]
fn invalid_block_finalized_slot() {
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.finalized_checkpoint.epoch == 0)
.apply_blocks(1)
.apply_invalid_block_directly_to_fork_choice(
|block, _| {
block.slot = Epoch::new(2).start_slot(E::slots_per_epoch()) - 1;
},
|err| {
assert_invalid_block!(
err,
InvalidBlock::FinalizedSlot { finalized_slot, .. }
if finalized_slot == Epoch::new(2).start_slot(E::slots_per_epoch())
)
},
);
}
/// Specification v0.12.1
///
/// assert get_ancestor(store, hash_tree_root(block), finalized_slot) ==
/// store.finalized_checkpoint.root
///
/// Note: we technically don't do this exact check, but an equivalent check. Reference:
///
/// https://github.com/ethereum/eth2.0-specs/pull/1884
#[test]
fn invalid_block_finalized_descendant() {
let invalid_ancestor = Mutex::new(Hash256::zero());
ForkChoiceTest::new()
.apply_blocks_while(|_, state| state.finalized_checkpoint.epoch == 0)
.apply_blocks(1)
.assert_finalized_epoch(2)
.apply_invalid_block_directly_to_fork_choice(
|block, state| {
block.parent_root = *state
.get_block_root(Epoch::new(1).start_slot(E::slots_per_epoch()))
.unwrap();
*invalid_ancestor.lock().unwrap() = block.parent_root;
},
|err| {
assert_invalid_block!(
err,
InvalidBlock::NotFinalizedDescendant { block_ancestor, .. }
if block_ancestor == Some(*invalid_ancestor.lock().unwrap())
)
},
);
}
macro_rules! assert_invalid_attestation {
($err: tt, $($error: pat) |+ $( if $guard: expr )?) => {
assert!(
matches!(
$err,
$( Err(BeaconChainError::ForkChoiceError(ForkChoiceError::InvalidAttestation($error))) ) |+ $( if $guard )?
),
"{:?}",
$err
);
};
}
/// Ensure we can process a valid attestation.
#[test]
fn valid_attestation() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|_, _| {},
|result| assert_eq!(result.unwrap(), ()),
);
}
/// This test is not in the specification, however we reject an attestation with an empty
/// aggregation bitfield since it has no purpose beyond wasting our time.
#[test]
fn invalid_attestation_empty_bitfield() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, _| {
attestation.attesting_indices = vec![].into();
},
|result| {
assert_invalid_attestation!(result, InvalidAttestation::EmptyAggregationBitfield)
},
);
}
/// Specification v0.12.1:
///
/// assert target.epoch in [expected_current_epoch, previous_epoch]
///
/// (tests epoch after current epoch)
#[test]
fn invalid_attestation_future_epoch() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, _| {
attestation.data.target.epoch = Epoch::new(2);
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::FutureEpoch { attestation_epoch, current_epoch }
if attestation_epoch == Epoch::new(2) && current_epoch == Epoch::new(0)
)
},
);
}
/// Specification v0.12.1:
///
/// assert target.epoch in [expected_current_epoch, previous_epoch]
///
/// (tests epoch prior to previous epoch)
#[test]
fn invalid_attestation_past_epoch() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(E::slots_per_epoch() as usize * 3 + 1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, _| {
attestation.data.target.epoch = Epoch::new(0);
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::PastEpoch { attestation_epoch, current_epoch }
if attestation_epoch == Epoch::new(0) && current_epoch == Epoch::new(3)
)
},
);
}
/// Specification v0.12.1:
///
/// assert target.epoch == compute_epoch_at_slot(attestation.data.slot)
#[test]
fn invalid_attestation_target_epoch() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(E::slots_per_epoch() as usize + 1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, _| {
attestation.data.slot = Slot::new(1);
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::BadTargetEpoch { target, slot }
if target == Epoch::new(1) && slot == Slot::new(1)
)
},
);
}
/// Specification v0.12.1:
///
/// assert target.root in store.blocks
#[test]
fn invalid_attestation_unknown_target_root() {
let junk = Hash256::from_low_u64_be(42);
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, _| {
attestation.data.target.root = junk;
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::UnknownTargetRoot(root)
if root == junk
)
},
);
}
/// Specification v0.12.1:
///
/// assert attestation.data.beacon_block_root in store.blocks
#[test]
fn invalid_attestation_unknown_beacon_block_root() {
let junk = Hash256::from_low_u64_be(42);
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, _| {
attestation.data.beacon_block_root = junk;
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::UnknownHeadBlock { beacon_block_root }
if beacon_block_root == junk
)
},
);
}
/// Specification v0.12.1:
///
/// assert store.blocks[attestation.data.beacon_block_root].slot <= attestation.data.slot
#[test]
fn invalid_attestation_future_block() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::Blocks(1),
|attestation, chain| {
attestation.data.beacon_block_root = chain
.block_at_slot(chain.slot().unwrap())
.unwrap()
.unwrap()
.canonical_root();
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::AttestsToFutureBlock { block, attestation }
if block == 2 && attestation == 1
)
},
);
}
/// Specification v0.12.1:
///
/// assert target.root == get_ancestor(store, attestation.data.beacon_block_root, target_slot)
#[test]
fn invalid_attestation_inconsistent_ffg_vote() {
let local_opt = Mutex::new(None);
let attestation_opt = Mutex::new(None);
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|attestation, chain| {
attestation.data.target.root = chain
.block_at_slot(Slot::new(1))
.unwrap()
.unwrap()
.canonical_root();
*attestation_opt.lock().unwrap() = Some(attestation.data.target.root);
*local_opt.lock().unwrap() = Some(
chain
.block_at_slot(Slot::new(0))
.unwrap()
.unwrap()
.canonical_root(),
);
},
|result| {
assert_invalid_attestation!(
result,
InvalidAttestation::InvalidTarget { attestation, local }
if attestation == attestation_opt.lock().unwrap().unwrap()
&& local == local_opt.lock().unwrap().unwrap()
)
},
);
}
/// Specification v0.12.1:
///
/// assert get_current_slot(store) >= attestation.data.slot + 1
#[test]
fn invalid_attestation_delayed_slot() {
ForkChoiceTest::new()
.apply_blocks_without_new_attestations(1)
.inspect_queued_attestations(|queue| assert_eq!(queue.len(), 0))
.apply_attestation_to_chain(
MutationDelay::NoDelay,
|_, _| {},
|result| assert_eq!(result.unwrap(), ()),
)
.inspect_queued_attestations(|queue| assert_eq!(queue.len(), 1))
.skip_slot()
.inspect_queued_attestations(|queue| assert_eq!(queue.len(), 0));
}

View File

@ -1,15 +1,14 @@
[package]
name = "proto_array_fork_choice"
name = "proto_array"
version = "0.2.0"
authors = ["Paul Hauner <paul@sigmaprime.io>"]
edition = "2018"
[[bin]]
name = "proto_array_fork_choice"
name = "proto_array"
path = "src/bin.rs"
[dependencies]
parking_lot = "0.10.2"
types = { path = "../types" }
itertools = "0.9.0"
eth2_ssz = "0.1.2"

View File

@ -1,4 +1,4 @@
use proto_array_fork_choice::fork_choice_test_definition::*;
use proto_array::fork_choice_test_definition::*;
use serde_yaml;
use std::fs::File;

View File

@ -2,7 +2,7 @@ mod ffg_updates;
mod no_votes;
mod votes;
use crate::proto_array_fork_choice::ProtoArrayForkChoice;
use crate::proto_array_fork_choice::{Block, ProtoArrayForkChoice};
use serde_derive::{Deserialize, Serialize};
use types::{Epoch, Hash256, Slot};
@ -55,7 +55,7 @@ pub struct ForkChoiceTestDefinition {
impl ForkChoiceTestDefinition {
pub fn run(self) {
let fork_choice = ProtoArrayForkChoice::new(
let mut fork_choice = ProtoArrayForkChoice::new(
self.finalized_block_slot,
Hash256::zero(),
self.justified_epoch,
@ -120,19 +120,19 @@ impl ForkChoiceTestDefinition {
justified_epoch,
finalized_epoch,
} => {
fork_choice
.process_block(
slot,
root,
parent_root,
Hash256::zero(),
justified_epoch,
finalized_epoch,
)
.expect(&format!(
"process_block op at index {} returned error",
op_index
));
let block = Block {
slot,
root,
parent_root: Some(parent_root),
state_root: Hash256::zero(),
target_root: Hash256::zero(),
justified_epoch,
finalized_epoch,
};
fork_choice.process_block(block).expect(&format!(
"process_block op at index {} returned error",
op_index
));
check_bytes_round_trip(&fork_choice);
}
Operation::ProcessAttestation {

View File

@ -4,7 +4,7 @@ mod proto_array;
mod proto_array_fork_choice;
mod ssz_container;
pub use crate::proto_array_fork_choice::ProtoArrayForkChoice;
pub use crate::proto_array_fork_choice::{Block, ProtoArrayForkChoice};
pub use error::Error;
pub mod core {

View File

@ -1,4 +1,4 @@
use crate::error::Error;
use crate::{error::Error, Block};
use serde_derive::{Deserialize, Serialize};
use ssz_derive::{Decode, Encode};
use std::collections::HashMap;
@ -12,10 +12,16 @@ pub struct ProtoNode {
/// The `state_root` is not necessary for `ProtoArray` either, it also just exists for upstream
/// components (namely attestation verification).
pub state_root: Hash256,
root: Hash256,
parent: Option<usize>,
justified_epoch: Epoch,
finalized_epoch: Epoch,
/// The root that would be used for the `attestation.data.target.root` if a LMD vote was cast
/// for this block.
///
/// The `target_root` is not necessary for `ProtoArray` either, it also just exists for upstream
/// components (namely fork choice attestation verification).
pub target_root: Hash256,
pub root: Hash256,
pub parent: Option<usize>,
pub justified_epoch: Epoch,
pub finalized_epoch: Epoch,
weight: u64,
best_child: Option<usize>,
best_descendant: Option<usize>,
@ -124,29 +130,24 @@ impl ProtoArray {
/// Register a block with the fork choice.
///
/// It is only sane to supply a `None` parent for the genesis block.
pub fn on_block(
&mut self,
slot: Slot,
root: Hash256,
parent_opt: Option<Hash256>,
state_root: Hash256,
justified_epoch: Epoch,
finalized_epoch: Epoch,
) -> Result<(), Error> {
pub fn on_block(&mut self, block: Block) -> Result<(), Error> {
// If the block is already known, simply ignore it.
if self.indices.contains_key(&root) {
if self.indices.contains_key(&block.root) {
return Ok(());
}
let node_index = self.nodes.len();
let node = ProtoNode {
slot,
state_root,
root,
parent: parent_opt.and_then(|parent| self.indices.get(&parent).copied()),
justified_epoch,
finalized_epoch,
slot: block.slot,
root: block.root,
target_root: block.target_root,
state_root: block.state_root,
parent: block
.parent_root
.and_then(|parent| self.indices.get(&parent).copied()),
justified_epoch: block.justified_epoch,
finalized_epoch: block.finalized_epoch,
weight: 0,
best_child: None,
best_descendant: None,

View File

@ -1,11 +1,9 @@
use crate::error::Error;
use crate::proto_array::ProtoArray;
use crate::ssz_container::SszContainer;
use parking_lot::{RwLock, RwLockReadGuard};
use ssz::{Decode, Encode};
use ssz_derive::{Decode, Encode};
use std::collections::HashMap;
use std::ptr;
use types::{Epoch, Hash256, Slot};
pub const DEFAULT_PRUNE_THRESHOLD: usize = 256;
@ -17,6 +15,19 @@ pub struct VoteTracker {
next_epoch: Epoch,
}
/// A block that is to be applied to the fork choice.
///
/// A simplified version of `types::BeaconBlock`.
pub struct Block {
pub slot: Slot,
pub root: Hash256,
pub parent_root: Option<Hash256>,
pub state_root: Hash256,
pub target_root: Hash256,
pub justified_epoch: Epoch,
pub finalized_epoch: Epoch,
}
/// A Vec-wrapper which will grow to match any request.
///
/// E.g., a `get` or `insert` to an out-of-bounds element will cause the Vec to grow (using
@ -44,21 +55,11 @@ where
}
}
#[derive(PartialEq)]
pub struct ProtoArrayForkChoice {
pub(crate) proto_array: RwLock<ProtoArray>,
pub(crate) votes: RwLock<ElasticList<VoteTracker>>,
pub(crate) balances: RwLock<Vec<u64>>,
}
impl PartialEq for ProtoArrayForkChoice {
fn eq(&self, other: &Self) -> bool {
if ptr::eq(self, other) {
return true;
}
*self.proto_array.read() == *other.proto_array.read()
&& *self.votes.read() == *other.votes.read()
&& *self.balances.read() == *other.balances.read()
}
pub(crate) proto_array: ProtoArray,
pub(crate) votes: ElasticList<VoteTracker>,
pub(crate) balances: Vec<u64>,
}
impl ProtoArrayForkChoice {
@ -77,32 +78,36 @@ impl ProtoArrayForkChoice {
indices: HashMap::with_capacity(1),
};
let block = Block {
slot: finalized_block_slot,
root: finalized_root,
parent_root: None,
state_root: finalized_block_state_root,
// We are using the finalized_root as the target_root, since it always lies on an
// epoch boundary.
target_root: finalized_root,
justified_epoch,
finalized_epoch,
};
proto_array
.on_block(
finalized_block_slot,
finalized_root,
None,
finalized_block_state_root,
justified_epoch,
finalized_epoch,
)
.on_block(block)
.map_err(|e| format!("Failed to add finalized block to proto_array: {:?}", e))?;
Ok(Self {
proto_array: RwLock::new(proto_array),
votes: RwLock::new(ElasticList::default()),
balances: RwLock::new(vec![]),
proto_array: proto_array,
votes: ElasticList::default(),
balances: vec![],
})
}
pub fn process_attestation(
&self,
&mut self,
validator_index: usize,
block_root: Hash256,
target_epoch: Epoch,
) -> Result<(), String> {
let mut votes = self.votes.write();
let vote = votes.get_mut(validator_index);
let vote = self.votes.get_mut(validator_index);
if target_epoch > vote.next_epoch || *vote == VoteTracker::default() {
vote.next_root = block_root;
@ -112,102 +117,86 @@ impl ProtoArrayForkChoice {
Ok(())
}
pub fn process_block(
&self,
slot: Slot,
block_root: Hash256,
parent_root: Hash256,
state_root: Hash256,
justified_epoch: Epoch,
finalized_epoch: Epoch,
) -> Result<(), String> {
pub fn process_block(&mut self, block: Block) -> Result<(), String> {
if block.parent_root.is_none() {
return Err("Missing parent root".to_string());
}
self.proto_array
.write()
.on_block(
slot,
block_root,
Some(parent_root),
state_root,
justified_epoch,
finalized_epoch,
)
.on_block(block)
.map_err(|e| format!("process_block_error: {:?}", e))
}
pub fn find_head(
&self,
&mut self,
justified_epoch: Epoch,
justified_root: Hash256,
finalized_epoch: Epoch,
justified_state_balances: &[u64],
) -> Result<Hash256, String> {
let mut proto_array = self.proto_array.write();
let mut votes = self.votes.write();
let mut old_balances = self.balances.write();
let old_balances = &mut self.balances;
let new_balances = justified_state_balances;
let deltas = compute_deltas(
&proto_array.indices,
&mut votes,
&self.proto_array.indices,
&mut self.votes,
&old_balances,
&new_balances,
)
.map_err(|e| format!("find_head compute_deltas failed: {:?}", e))?;
proto_array
self.proto_array
.apply_score_changes(deltas, justified_epoch, finalized_epoch)
.map_err(|e| format!("find_head apply_score_changes failed: {:?}", e))?;
*old_balances = new_balances.to_vec();
proto_array
self.proto_array
.find_head(&justified_root)
.map_err(|e| format!("find_head failed: {:?}", e))
}
pub fn maybe_prune(&self, finalized_root: Hash256) -> Result<(), String> {
pub fn maybe_prune(&mut self, finalized_root: Hash256) -> Result<(), String> {
self.proto_array
.write()
.maybe_prune(finalized_root)
.map_err(|e| format!("find_head maybe_prune failed: {:?}", e))
}
pub fn set_prune_threshold(&self, prune_threshold: usize) {
self.proto_array.write().prune_threshold = prune_threshold;
pub fn set_prune_threshold(&mut self, prune_threshold: usize) {
self.proto_array.prune_threshold = prune_threshold;
}
pub fn len(&self) -> usize {
self.proto_array.read().nodes.len()
self.proto_array.nodes.len()
}
pub fn contains_block(&self, block_root: &Hash256) -> bool {
self.proto_array.read().indices.contains_key(block_root)
self.proto_array.indices.contains_key(block_root)
}
pub fn block_slot(&self, block_root: &Hash256) -> Option<Slot> {
let proto_array = self.proto_array.read();
pub fn get_block(&self, block_root: &Hash256) -> Option<Block> {
let block_index = self.proto_array.indices.get(block_root)?;
let block = self.proto_array.nodes.get(*block_index)?;
let parent_root = block
.parent
.and_then(|i| self.proto_array.nodes.get(i))
.map(|parent| parent.root);
let i = proto_array.indices.get(block_root)?;
let block = proto_array.nodes.get(*i)?;
Some(block.slot)
}
pub fn block_slot_and_state_root(&self, block_root: &Hash256) -> Option<(Slot, Hash256)> {
let proto_array = self.proto_array.read();
let i = proto_array.indices.get(block_root)?;
let block = proto_array.nodes.get(*i)?;
Some((block.slot, block.state_root))
Some(Block {
slot: block.slot,
root: block.root,
parent_root,
state_root: block.state_root,
target_root: block.target_root,
justified_epoch: block.justified_epoch,
finalized_epoch: block.finalized_epoch,
})
}
pub fn latest_message(&self, validator_index: usize) -> Option<(Hash256, Epoch)> {
let votes = self.votes.read();
if validator_index < votes.0.len() {
let vote = &votes.0[validator_index];
if validator_index < self.votes.0.len() {
let vote = &self.votes.0[validator_index];
if *vote == VoteTracker::default() {
None
@ -232,8 +221,8 @@ impl ProtoArrayForkChoice {
/// Returns a read-lock to core `ProtoArray` struct.
///
/// Should only be used when encoding/decoding during troubleshooting.
pub fn core_proto_array(&self) -> RwLockReadGuard<ProtoArray> {
self.proto_array.read()
pub fn core_proto_array(&self) -> &ProtoArray {
&self.proto_array
}
}

View File

@ -2,7 +2,6 @@ use crate::{
proto_array::{ProtoArray, ProtoNode},
proto_array_fork_choice::{ElasticList, ProtoArrayForkChoice, VoteTracker},
};
use parking_lot::RwLock;
use ssz_derive::{Decode, Encode};
use std::collections::HashMap;
use std::iter::FromIterator;
@ -21,11 +20,11 @@ pub struct SszContainer {
impl From<&ProtoArrayForkChoice> for SszContainer {
fn from(from: &ProtoArrayForkChoice) -> Self {
let proto_array = from.proto_array.read();
let proto_array = &from.proto_array;
Self {
votes: from.votes.read().0.clone(),
balances: from.balances.read().clone(),
votes: from.votes.0.clone(),
balances: from.balances.clone(),
prune_threshold: proto_array.prune_threshold,
justified_epoch: proto_array.justified_epoch,
finalized_epoch: proto_array.finalized_epoch,
@ -46,9 +45,9 @@ impl From<SszContainer> for ProtoArrayForkChoice {
};
Self {
proto_array: RwLock::new(proto_array),
votes: RwLock::new(ElasticList(from.votes)),
balances: RwLock::new(from.balances),
proto_array: proto_array,
votes: ElasticList(from.votes),
balances: from.balances,
}
}
}

View File

@ -12,6 +12,7 @@ use tree_hash_derive::TreeHash;
#[derive(
Debug,
Clone,
Copy,
PartialEq,
Eq,
Default,