implement clock disparity for attestation validation (#1568)

This implements disparity, resolving a part of
https://github.com/status-im/nim-beacon-chain/issues/1367

* make BeaconTime a duration for fractional seconds
* factor out attestation/aggregate validation
* simplify recording of queued attestations
* simplify attestation signature check
* fix blocks_received metric
* add some trivial validation tests
* remove unresolved attestation table - attestations for unknown blocks
are dropped instead (cannot verify their signature)
This commit is contained in:
Jacek Sieka 2020-08-27 09:34:12 +02:00 committed by GitHub
parent 2081c4f505
commit fa1621db46
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
15 changed files with 386 additions and 340 deletions

View File

@ -13,6 +13,11 @@ AllTests-mainnet
+ Trying to add a duplicate block from an old pruned epoch is tagged as an error OK
```
OK: 9/9 Fail: 0/9 Skip: 0/9
## Attestation validation [Preset: mainnet]
```diff
+ Validation sanity OK
```
OK: 1/1 Fail: 0/1 Skip: 0/1
## Beacon chain DB [Preset: mainnet]
```diff
+ empty database [Preset: mainnet] OK
@ -248,4 +253,4 @@ OK: 1/1 Fail: 0/1 Skip: 0/1
OK: 1/1 Fail: 0/1 Skip: 0/1
---TOTAL---
OK: 135/142 Fail: 0/142 Skip: 7/142
OK: 136/143 Fail: 0/143 Skip: 7/143

View File

@ -13,11 +13,14 @@ import
beaconstate, datatypes, crypto, digest, helpers, network, validator,
signatures],
./block_pools/[spec_cache, chain_dag, quarantine, spec_cache],
./attestation_pool, ./beacon_node_types, ./ssz
./attestation_pool, ./beacon_node_types, ./ssz, ./time
logScope:
topics = "att_aggr"
const
MAXIMUM_GOSSIP_CLOCK_DISPARITY = 500.millis
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/validator.md#aggregation-selection
func is_aggregator(state: BeaconState, slot: Slot, index: CommitteeIndex,
slot_signature: ValidatorSig, cache: var StateCache): bool =
@ -57,75 +60,118 @@ proc aggregate_attestations*(
aggregate: maybe_slot_attestation.get,
selection_proof: slot_signature))
proc isValidAttestationSlot(
pool: AttestationPool, attestationSlot: Slot, attestationBlck: BlockRef): bool =
func check_attestation_block_slot(
pool: AttestationPool, attestationSlot: Slot, attestationBlck: BlockRef): Result[void, cstring] =
# If we allow voting for very old blocks, the state transaction below will go
# nuts and keep processing empty slots
logScope:
attestationSlot
attestationBlck = shortLog(attestationBlck)
if not (attestationBlck.slot > pool.chainDag.finalizedHead.slot):
debug "Voting for already-finalized block"
return false
return err("Voting for already-finalized block")
# we'll also cap it at 4 epochs which is somewhat arbitrary, but puts an
# upper bound on the processing done to validate the attestation
# TODO revisit with less arbitrary approach
if not (attestationSlot >= attestationBlck.slot):
debug "Voting for block that didn't exist at the time"
return false
return err("Voting for block that didn't exist at the time")
if not ((attestationSlot - attestationBlck.slot) <= uint64(4 * SLOTS_PER_EPOCH)):
debug "Voting for very old block"
return false
return err("Voting for very old block")
true
ok()
func checkPropagationSlotRange(data: AttestationData, current_slot: Slot): bool =
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/p2p-interface.md#beacon_attestation_subnet_id
# TODO clock disparity of 0.5s instead of whole slot
# attestation.data.slot + ATTESTATION_PROPAGATION_SLOT_RANGE >=
# current_slot >= attestation.data.slot
(data.slot + ATTESTATION_PROPAGATION_SLOT_RANGE + 1 >= current_slot) and
(current_slot >= data.slot)
func check_propagation_slot_range(
data: AttestationData, wallTime: BeaconTime): Result[void, cstring] =
let
futureSlot = (wallTime + MAXIMUM_GOSSIP_CLOCK_DISPARITY).toSlot()
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/p2p-interface.md#beacon_attestation_subnet_id
proc isValidAttestation*(
pool: var AttestationPool,
attestation: Attestation, current_slot: Slot,
topicCommitteeIndex: uint64): bool =
logScope:
topics = "att_aggr valid_att"
received_attestation = shortLog(attestation)
if not futureSlot.afterGenesis or data.slot > futureSlot.slot:
return err("Attestation slot in the future")
# TODO https://github.com/ethereum/eth2.0-specs/issues/1998
if (let v = check_attestation_slot_target(attestation.data); v.isErr):
debug "Invalid attestation", err = v.error
return false
let
pastSlot = (wallTime - MAXIMUM_GOSSIP_CLOCK_DISPARITY).toSlot()
if not checkPropagationSlotRange(attestation.data, current_slot):
debug "Attestation slot not in propagation range",
current_slot
return false
if pastSlot.afterGenesis and
data.slot + ATTESTATION_PROPAGATION_SLOT_RANGE < pastSlot.slot:
return err("Attestation slot in the past")
# The attestation is unaggregated -- that is, it has exactly one
# participating validator (len([bit for bit in attestation.aggregation_bits
# if bit == 0b1]) == 1).
ok()
func check_attestation_beacon_block(
pool: var AttestationPool, attestation: Attestation): Result[void, cstring] =
# The block being voted for (attestation.data.beacon_block_root) passes
# validation.
# We rely on the chain DAG to have been validated, so check for the existence
# of the block in the pool.
let attestationBlck = pool.chainDag.getRef(attestation.data.beacon_block_root)
if attestationBlck.isNil:
pool.quarantine.addMissing(attestation.data.beacon_block_root)
return err("Attestation block unknown")
# Not in spec - check that rewinding to the state is sane
? check_attestation_block_slot(pool, attestation.data.slot, attestationBlck)
ok()
func check_aggregation_count(
attestation: Attestation, singular: bool): Result[void, cstring] =
var onesCount = 0
# TODO a cleverer algorithm, along the lines of countOnes() in nim-stew
# But that belongs in nim-stew, since it'd break abstraction layers, to
# use details of its representation from nim-beacon-chain.
var onesCount = 0
for aggregation_bit in attestation.aggregation_bits:
if not aggregation_bit:
continue
onesCount += 1
if onesCount > 1:
debug "Attestation has too many aggregation bits"
return false
if onesCount != 1:
debug "Attestation has too few aggregation bits"
return false
if singular: # More than one ok
if onesCount > 1:
return err("Attestation has too many aggregation bits")
else:
break # Found the one we needed
if onesCount < 1:
return err("Attestation has too few aggregation bits")
ok()
func check_attestation_subnet(
epochRef: EpochRef, attestation: Attestation,
topicCommitteeIndex: uint64): Result[void, cstring] =
let
expectedSubnet =
compute_subnet_for_attestation(
get_committee_count_per_slot(epochRef),
attestation.data.slot, attestation.data.index.CommitteeIndex)
if expectedSubnet != topicCommitteeIndex:
return err("Attestation's committee index not for the correct subnet")
ok()
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/p2p-interface.md#beacon_attestation_subnet_id
proc validateAttestation*(
pool: var AttestationPool,
attestation: Attestation, wallTime: BeaconTime,
topicCommitteeIndex: uint64): Result[HashSet[ValidatorIndex], cstring] =
? check_attestation_slot_target(attestation.data) # Not in spec - ignore
# attestation.data.slot is within the last ATTESTATION_PROPAGATION_SLOT_RANGE
# slots (within a MAXIMUM_GOSSIP_CLOCK_DISPARITY allowance) -- i.e.
# attestation.data.slot + ATTESTATION_PROPAGATION_SLOT_RANGE >= current_slot
# >= attestation.data.slot (a client MAY queue future attestations for\
# processing at the appropriate slot).
? check_propagation_slot_range(attestation.data, wallTime) # [IGNORE]
# The attestation is unaggregated -- that is, it has exactly one
# participating validator (len([bit for bit in attestation.aggregation_bits
# if bit == 0b1]) == 1).
? check_aggregation_count(attestation, singular = true) # [REJECT]
# The block being voted for (attestation.data.beacon_block_root) has been seen
# (via both gossip and non-gossip sources) (a client MAY queue aggregates for
# processing once block is retrieved).
# The block being voted for (attestation.data.beacon_block_root) passes
# validation.
? check_attestation_beacon_block(pool, attestation) # [IGNORE/REJECT]
# The attestation is the first valid attestation received for the
# participating validator for the slot, attestation.data.slot.
@ -137,31 +183,12 @@ proc isValidAttestation*(
# Attestations might be aggregated eagerly or lazily; allow for both.
for validation in attestationEntry.validations:
if attestation.aggregation_bits.isSubsetOf(validation.aggregation_bits):
debug "Attestation already exists at slot",
attestation_pool_validation = validation.aggregation_bits
return false
# The block being voted for (attestation.data.beacon_block_root) passes
# validation.
# We rely on the chain DAG to have been validated, so check for the existence
# of the block in the pool.
let attestationBlck = pool.chainDag.getRef(attestation.data.beacon_block_root)
if attestationBlck.isNil:
debug "Attestation block unknown"
pool.addUnresolved(attestation)
pool.quarantine.addMissing(attestation.data.beacon_block_root)
return false
if not isValidAttestationSlot(pool, attestation.data.slot, attestationBlck):
# Not in spec - check that rewinding to the state is sane
return false
return err("Attestation already exists at slot") # [IGNORE]
let tgtBlck = pool.chainDag.getRef(attestation.data.target.root)
if tgtBlck.isNil:
debug "Attestation target block unknown"
pool.addUnresolved(attestation)
pool.quarantine.addMissing(attestation.data.target.root)
return false
return err("Attestation target block unknown")
# The following rule follows implicitly from that we clear out any
# unviable blocks from the chain dag:
@ -175,68 +202,43 @@ proc isValidAttestation*(
let epochRef = pool.chainDag.getEpochRef(
tgtBlck, attestation.data.target.epoch)
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/p2p-interface.md#beacon_attestation_subnet_id
# [REJECT] The attestation is for the correct subnet -- i.e.
# compute_subnet_for_attestation(committees_per_slot,
# attestation.data.slot, attestation.data.index) == subnet_id, where
# committees_per_slot = get_committee_count_per_slot(state,
# attestation.data.target.epoch), which may be pre-computed along with the
# committee information for the signature check.
let
requiredSubnetIndex =
compute_subnet_for_attestation(
get_committee_count_per_slot(epochRef),
attestation.data.slot, attestation.data.index.CommitteeIndex)
if requiredSubnetIndex != topicCommitteeIndex:
debug "Attestation's committee index not for the correct subnet",
topicCommitteeIndex = topicCommitteeIndex,
attestation_data_index = attestation.data.index,
requiredSubnetIndex = requiredSubnetIndex
return false
? check_attestation_subnet(epochRef, attestation, topicCommitteeIndex)
let
fork = pool.chainDag.headState.data.data.fork
genesis_validators_root =
pool.chainDag.headState.data.data.genesis_validators_root
attesting_indices = get_attesting_indices(
epochRef, attestation.data, attestation.aggregation_bits)
# The signature of attestation is valid.
if (let v = is_valid_indexed_attestation(
fork, genesis_validators_root,
epochRef, get_indexed_attestation(epochRef, attestation), {}); v.isErr):
debug "Attestation verification failed", err = v.error()
return false
? is_valid_indexed_attestation(
fork, genesis_validators_root, epochRef, attesting_indices, attestation, {})
true
ok(attesting_indices)
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/p2p-interface.md#beacon_aggregate_and_proof
proc isValidAggregatedAttestation*(
proc validateAggregate*(
pool: var AttestationPool,
signedAggregateAndProof: SignedAggregateAndProof,
current_slot: Slot): bool =
wallTime: BeaconTime): Result[HashSet[ValidatorIndex], cstring] =
let
aggregate_and_proof = signedAggregateAndProof.message
aggregate = aggregate_and_proof.aggregate
logScope:
aggregate = shortLog(aggregate)
? check_attestation_slot_target(aggregate.data) # Not in spec - ignore
# TODO https://github.com/ethereum/eth2.0-specs/issues/1998
if (let v = check_attestation_slot_target(aggregate.data); v.isErr):
debug "Invalid aggregate", err = v.error
return false
# There's some overlap between this and isValidAttestation(), but unclear if
# saving a few lines of code would balance well with losing straightforward,
# spec-based synchronization.
#
# [IGNORE] aggregate.data.slot is within the last
# ATTESTATION_PROPAGATION_SLOT_RANGE slots (with a
# MAXIMUM_GOSSIP_CLOCK_DISPARITY allowance) -- i.e. aggregate.data.slot +
# ATTESTATION_PROPAGATION_SLOT_RANGE >= current_slot >= aggregate.data.slot
if not checkPropagationSlotRange(aggregate.data, current_slot):
debug "Aggregate slot not in propagation range"
return false
? check_propagation_slot_range(aggregate.data, wallTime) # [IGNORE]
# [IGNORE] The valid aggregate attestation defined by
# hash_tree_root(aggregate) has not already been seen (via aggregate gossip,
@ -253,14 +255,6 @@ proc isValidAggregatedAttestation*(
# This is [IGNORE] and already effectively checked by attestation pool upon
# attempting to resolve attestations.
# [REJECT] The block being voted for (aggregate.data.beacon_block_root)
# passes validation.
let attestationBlck = pool.chainDag.getRef(aggregate.data.beacon_block_root)
if attestationBlck.isNil:
debug "Aggregate block unknown"
pool.quarantine.addMissing(aggregate.data.beacon_block_root)
return false
# [REJECT] The attestation has participants -- that is,
# len(get_attesting_indices(state, aggregate.data, aggregate.aggregation_bits)) >= 1.
#
@ -274,30 +268,26 @@ proc isValidAggregatedAttestation*(
# members, i.e. they counts don't match.
# But (2) would reflect an invalid aggregation in other ways, so reject it
# either way.
if isZeros(aggregate.aggregation_bits):
debug "Aggregate has no or invalid aggregation bits"
return false
? check_aggregation_count(aggregate, singular = false)
if not isValidAttestationSlot(pool, aggregate.data.slot, attestationBlck):
# Not in spec - check that rewinding to the state is sane
return false
# [REJECT] The block being voted for (aggregate.data.beacon_block_root)
# passes validation.
? check_attestation_beacon_block(pool, aggregate)
# [REJECT] aggregate_and_proof.selection_proof selects the validator as an
# aggregator for the slot -- i.e. is_aggregator(state, aggregate.data.slot,
# aggregate.data.index, aggregate_and_proof.selection_proof) returns True.
let tgtBlck = pool.chainDag.getRef(aggregate.data.target.root)
if tgtBlck.isNil:
debug "Aggregate target block unknown"
pool.quarantine.addMissing(aggregate.data.target.root)
return
return err("Aggregate target block unknown")
let epochRef = pool.chainDag.getEpochRef(tgtBlck, aggregate.data.target.epoch)
if not is_aggregator(
epochRef, aggregate.data.slot, aggregate.data.index.CommitteeIndex,
aggregate_and_proof.selection_proof):
debug "Incorrect aggregator"
return false
return err("Incorrect aggregator")
# [REJECT] The aggregator's validator index is within the committee -- i.e.
# aggregate_and_proof.aggregator_index in get_beacon_committee(state,
@ -305,16 +295,14 @@ proc isValidAggregatedAttestation*(
if aggregate_and_proof.aggregator_index.ValidatorIndex notin
get_beacon_committee(
epochRef, aggregate.data.slot, aggregate.data.index.CommitteeIndex):
debug "Aggregator's validator index not in committee"
return false
return err("Aggregator's validator index not in committee")
# [REJECT] The aggregate_and_proof.selection_proof is a valid signature of the
# aggregate.data.slot by the validator with index
# aggregate_and_proof.aggregator_index.
# get_slot_signature(state, aggregate.data.slot, privkey)
if aggregate_and_proof.aggregator_index >= epochRef.validator_keys.lenu64:
debug "Invalid aggregator_index"
return false
return err("Invalid aggregator_index")
let
fork = pool.chainDag.headState.data.data.fork
@ -324,23 +312,21 @@ proc isValidAggregatedAttestation*(
fork, genesis_validators_root, aggregate.data.slot,
epochRef.validator_keys[aggregate_and_proof.aggregator_index],
aggregate_and_proof.selection_proof):
debug "Selection_proof signature verification failed"
return false
return err("Selection_proof signature verification failed")
# [REJECT] The aggregator signature, signed_aggregate_and_proof.signature, is valid.
if not verify_aggregate_and_proof_signature(
fork, genesis_validators_root, aggregate_and_proof,
epochRef.validator_keys[aggregate_and_proof.aggregator_index],
signed_aggregate_and_proof.signature):
debug "signed_aggregate_and_proof signature verification failed"
return false
return err("signed_aggregate_and_proof signature verification failed")
let attesting_indices = get_attesting_indices(
epochRef, aggregate.data, aggregate.aggregation_bits)
# [REJECT] The signature of aggregate is valid.
if (let v = is_valid_indexed_attestation(
fork, genesis_validators_root,
epochRef, get_indexed_attestation(epochRef, aggregate), {}); v.isErr):
debug "Aggregate verification failed", err = v.error()
return false
? is_valid_indexed_attestation(
fork, genesis_validators_root, epochRef, attesting_indices, aggregate, {})
# The following rule follows implicitly from that we clear out any
# unviable blocks from the chain dag:
@ -351,4 +337,4 @@ proc isValidAggregatedAttestation*(
# compute_start_slot_at_epoch(store.finalized_checkpoint.epoch)) ==
# store.finalized_checkpoint.root
true
ok(attesting_indices)

View File

@ -25,9 +25,6 @@ proc init*(T: type AttestationPool, chainDag: ChainDAGRef, quarantine: Quarantin
## Initialize an AttestationPool from the chainDag `headState`
## The `finalized_root` works around the finalized_checkpoint of the genesis block
## holding a zero_root.
# TODO chainDag/quarantine are only used when resolving orphaned attestations - they
# should probably be removed as a dependency of AttestationPool (or some other
# smart refactoring)
let
finalizedEpochRef = chainDag.getEpochRef(
@ -65,25 +62,18 @@ proc init*(T: type AttestationPool, chainDag: ChainDAGRef, quarantine: Quarantin
T(
chainDag: chainDag,
quarantine: quarantine,
unresolved: initTable[Eth2Digest, UnresolvedAttestation](),
forkChoice: forkChoice
)
proc processAttestation(
pool: var AttestationPool, slot: Slot, participants: HashSet[ValidatorIndex],
block_root: Eth2Digest, target: Checkpoint, wallSlot: Slot) =
block_root: Eth2Digest, wallSlot: Slot) =
# Add attestation votes to fork choice
if (let v = pool.forkChoice.on_attestation(
pool.chainDag, slot, block_root, toSeq(participants), target, wallSlot);
pool.chainDag, slot, block_root, participants, wallSlot);
v.isErr):
warn "Couldn't process attestation", err = v.error()
func addUnresolved*(pool: var AttestationPool, attestation: Attestation) =
pool.unresolved[attestation.data.beacon_block_root] =
UnresolvedAttestation(
attestation: attestation,
)
func candidateIdx(pool: AttestationPool, slot: Slot): Option[uint64] =
if slot >= pool.startingSlot and
slot < (pool.startingSlot + pool.candidates.lenu64):
@ -109,17 +99,16 @@ proc updateCurrent(pool: var AttestationPool, wallSlot: Slot) =
pool.startingSlot = newWallSlot
proc addResolved(
pool: var AttestationPool, blck: BlockRef, attestation: Attestation,
wallSlot: Slot) =
proc addAttestation*(pool: var AttestationPool,
attestation: Attestation,
participants: HashSet[ValidatorIndex],
wallSlot: Slot) =
# Add an attestation whose parent we know
logScope:
attestation = shortLog(attestation)
updateCurrent(pool, wallSlot)
doAssert blck.root == attestation.data.beacon_block_root
let candidateIdx = pool.candidateIdx(attestation.data.slot)
if candidateIdx.isNone:
debug "Attestation slot out of range",
@ -127,13 +116,10 @@ proc addResolved(
return
let
epochRef = pool.chainDag.getEpochRef(blck, attestation.data.target.epoch)
attestationsSeen = addr pool.candidates[candidateIdx.get]
validation = Validation(
aggregation_bits: attestation.aggregation_bits,
aggregate_signature: attestation.signature)
participants = get_attesting_indices(
epochRef, attestation.data, validation.aggregation_bits)
var found = false
for a in attestationsSeen.attestations.mitems():
@ -164,12 +150,11 @@ proc addResolved(
a.validations.add(validation)
pool.processAttestation(
attestation.data.slot, participants, attestation.data.beacon_block_root,
attestation.data.target, wallSlot)
wallSlot)
info "Attestation resolved",
attestation = shortLog(attestation),
validations = a.validations.len(),
blockSlot = shortLog(blck.slot)
validations = a.validations.len()
found = true
@ -178,36 +163,15 @@ proc addResolved(
if not found:
attestationsSeen.attestations.add(AttestationEntry(
data: attestation.data,
blck: blck,
validations: @[validation]
))
pool.processAttestation(
attestation.data.slot, participants, attestation.data.beacon_block_root,
attestation.data.target, wallSlot)
wallSlot)
info "Attestation resolved",
attestation = shortLog(attestation),
validations = 1,
blockSlot = shortLog(blck.slot)
proc addAttestation*(pool: var AttestationPool,
attestation: Attestation,
wallSlot: Slot) =
## Add a verified attestation to the fork choice context
logScope: pcs = "atp_add_attestation"
# Fetch the target block or notify the block pool that it's needed
let blck = pool.chainDag.getOrResolve(
pool.quarantine,
attestation.data.beacon_block_root)
# If the block exist, add it to the fork choice context
# Otherwise delay until it resolves
if blck.isNil:
pool.addUnresolved(attestation)
return
pool.addResolved(blck, attestation, wallSlot)
validations = 1
proc addForkChoice*(pool: var AttestationPool,
epochRef: EpochRef,
@ -261,7 +225,7 @@ proc getAttestationsForBlock*(pool: AttestationPool,
# intended thing -- sure, _blocks_ have to be popular (via attestation)
# but _attestations_ shouldn't have to be so frequently repeated, as an
# artifact of this state-free, identical-across-clones choice basis. In
# addResolved, too, the new attestations get added to the end, while in
# addAttestation, too, the new attestations get added to the end, while in
# these functions, it's reading from the beginning, et cetera. This all
# needs a single unified strategy.
for i in max(1, newBlockSlot.int64 - ATTESTATION_LOOKBACK.int64) .. newBlockSlot.int64:
@ -356,30 +320,6 @@ proc getAggregatedAttestation*(pool: AttestationPool,
none(Attestation)
proc resolve*(pool: var AttestationPool, wallSlot: Slot) =
## Check attestations in our unresolved deque
## if they can be integrated to the fork choice
logScope: pcs = "atp_resolve"
var
done: seq[Eth2Digest]
resolved: seq[tuple[blck: BlockRef, attestation: Attestation]]
for k, v in pool.unresolved.mpairs():
if (let blck = pool.chainDag.getRef(k); not blck.isNil()):
resolved.add((blck, v.attestation))
done.add(k)
elif v.tries > 8:
done.add(k)
else:
inc v.tries
for k in done:
pool.unresolved.del(k)
for a in resolved:
pool.addResolved(a.blck, a.attestation, wallSlot)
proc selectHead*(pool: var AttestationPool, wallSlot: Slot): BlockRef =
## Trigger fork choice and returns the new head block.
## Can return `nil`

View File

@ -867,8 +867,8 @@ proc start(node: BeaconNode) =
version = fullVersionStr,
nim = shortNimBanner(),
timeSinceFinalization =
int64(finalizedHead.slot.toBeaconTime()) -
int64(node.beaconClock.now()),
finalizedHead.slot.toBeaconTime() -
node.beaconClock.now(),
head = shortLog(head),
finalizedHead = shortLog(finalizedHead),
SLOTS_PER_EPOCH,

View File

@ -3,7 +3,7 @@
import
deques, tables,
stew/endians2,
spec/[datatypes, crypto, digest],
spec/[datatypes, crypto],
block_pools/block_pools_types,
fork_choice/fork_choice_types
@ -44,10 +44,6 @@ type
## TODO this could be a Table[AttestationData, seq[Validation] or something
## less naive
UnresolvedAttestation* = object
attestation*: Attestation
tries*: int
AttestationPool* = object
## The attestation pool keeps track of all attestations that potentially
## could be added to a block during block production.
@ -66,8 +62,6 @@ type
chainDag*: ChainDAGRef
quarantine*: QuarantineRef
unresolved*: Table[Eth2Digest, UnresolvedAttestation]
forkChoice*: ForkChoice
# #############################################

View File

@ -122,6 +122,31 @@ proc is_valid_indexed_attestation*(
ok()
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/beacon-chain.md#is_valid_indexed_attestation
proc is_valid_indexed_attestation*(
fork: Fork, genesis_validators_root: Eth2Digest,
epochRef: EpochRef, attesting_indices: HashSet[ValidatorIndex],
attestation: SomeAttestation, flags: UpdateFlags): Result[void, cstring] =
# This is a variation on `is_valid_indexed_attestation` that works directly
# with an attestation instead of first constructing an `IndexedAttestation`
# and then validating it - for the purpose of validating the signature, the
# order doesn't matter and we can proceed straight to validating the
# signature instead
if attesting_indices.len == 0:
return err("indexed_attestation: no attesting indices")
# Verify aggregate signature
if skipBLSValidation notin flags:
# TODO: fuse loops with blsFastAggregateVerify
let pubkeys = mapIt(
attesting_indices, epochRef.validator_keys[it])
if not verify_attestation_signature(
fork, genesis_validators_root, attestation.data,
pubkeys, attestation.signature):
return err("indexed attestation: signature verification failure")
ok()
func makeAttestationData*(
epochRef: EpochRef, bs: BlockSlot,
committee_index: uint64): AttestationData =

View File

@ -36,16 +36,18 @@ declareGauge beacon_head_root,
type
GetWallTimeFn* = proc(): BeaconTime {.gcsafe, raises: [Defect].}
Entry[T] = object
v*: T
SyncBlock* = object
blk*: SignedBeaconBlock
resfut*: Future[Result[void, BlockError]]
BlockEntry* = Entry[SyncBlock]
AttestationEntry* = Entry[Attestation]
AggregateEntry* = Entry[Attestation]
BlockEntry* = object
v*: SyncBlock
AttestationEntry* = object
v*: Attestation
attesting_indices*: HashSet[ValidatorIndex]
AggregateEntry* = AttestationEntry
Eth2Processor* = object
config*: BeaconNodeConf
@ -61,9 +63,6 @@ type
proc updateHead*(self: var Eth2Processor, wallSlot: Slot): BlockRef =
## Trigger fork choice and returns the new head block.
## Can return `nil`
# Check pending attestations - maybe we found some blocks for them
self.attestationPool[].resolve(wallSlot)
# Grab the new head according to our latest attestation data
let newHead = self.attestationPool[].selectHead(wallSlot)
if newHead.isNil():
@ -153,8 +152,9 @@ proc processAttestation(
error "Processing attestation before genesis, clock turned back?"
quit 1
debug "Processing attestation"
self.attestationPool[].addAttestation(entry.v, wallSlot)
trace "Processing attestation"
self.attestationPool[].addAttestation(
entry.v, entry.attesting_indices, wallSlot)
proc processAggregate(
self: var Eth2Processor, entry: AggregateEntry) =
@ -169,8 +169,9 @@ proc processAggregate(
error "Processing aggregate before genesis, clock turned back?"
quit 1
debug "Processing aggregate"
self.attestationPool[].addAttestation(entry.v, wallSlot)
trace "Processing aggregate"
self.attestationPool[].addAttestation(
entry.v, entry.attesting_indices, wallSlot)
proc processBlock(self: var Eth2Processor, entry: BlockEntry) =
logScope:
@ -266,17 +267,18 @@ proc blockValidator*(
if not blck.isOk:
return false
beacon_blocks_received.inc()
beacon_block_delay.observe(float(milliseconds(delay)) / 1000.0)
# Block passed validation - enqueue it for processing. The block processing
# queue is effectively unbounded as we use a freestanding task to enqueue
# the block - this is done so that when blocks arrive concurrently with
# sync, we don't lose the gossip blocks, but also don't block the gossip
# propagation of seemingly good blocks
debug "Block validated"
trace "Block validated"
traceAsyncErrors self.blocksQueue.addLast(
BlockEntry(v: SyncBlock(blk: signedBlock)))
beacon_block_delay.observe(float(milliseconds(delay)) / 1000.0)
true
proc attestationValidator*(
@ -299,9 +301,11 @@ proc attestationValidator*(
let delay = wallTime - attestation.data.slot.toBeaconTime
debug "Attestation received", delay
if not self.attestationPool[].isValidAttestation(
attestation, wallSlot, committeeIndex):
return false # logged in validation
let v = self.attestationPool[].validateAttestation(
attestation, wallTime, committeeIndex)
if v.isErr():
debug "Dropping attestation", err = v.error()
return false
beacon_attestations_received.inc()
beacon_attestation_delay.observe(float(milliseconds(delay)) / 1000.0)
@ -312,9 +316,9 @@ proc attestationValidator*(
notice "Queue full, dropping attestation",
dropped = shortLog(dropped.read().v)
debug "Attestation validated"
trace "Attestation validated"
traceAsyncErrors self.attestationsQueue.addLast(
AttestationEntry(v: attestation))
AttestationEntry(v: attestation, attesting_indices: v.get()))
true
@ -339,8 +343,10 @@ proc aggregateValidator*(
wallTime - signedAggregateAndProof.message.aggregate.data.slot.toBeaconTime
debug "Aggregate received", delay
if not self.attestationPool[].isValidAggregatedAttestation(
signedAggregateAndProof, wallSlot):
let v = self.attestationPool[].validateAggregate(
signedAggregateAndProof, wallTime)
if v.isErr:
debug "Dropping aggregate", err = v.error
return false
beacon_aggregates_received.inc()
@ -352,9 +358,10 @@ proc aggregateValidator*(
notice "Queue full, dropping aggregate",
dropped = shortLog(dropped.read().v)
debug "Aggregate validated"
trace "Aggregate validated"
traceAsyncErrors self.aggregatesQueue.addLast(AggregateEntry(
v: signedAggregateAndProof.message.aggregate))
v: signedAggregateAndProof.message.aggregate,
attesting_indices: v.get()))
true

View File

@ -9,7 +9,7 @@
import
# Standard library
std/[sets, tables, typetraits],
std/[sequtils, sets, tables, typetraits],
# Status libraries
stew/results, chronicles,
# Internal
@ -157,7 +157,7 @@ proc process_attestation_queue(self: var ForkChoice) =
for validator_index in attestation.attesting_indices:
self.backend.process_attestation(
validator_index, attestation.block_root,
attestation.target_epoch)
attestation.slot.epoch())
else:
keep.add attestation
@ -174,10 +174,9 @@ func contains*(self: ForkChoiceBackend, block_root: Eth2Digest): bool =
proc on_attestation*(
self: var ForkChoice,
dag: ChainDAGRef,
slot: Slot,
attestation_slot: Slot,
beacon_block_root: Eth2Digest,
attesting_indices: openArray[ValidatorIndex],
target: Checkpoint,
attesting_indices: HashSet[ValidatorIndex],
wallSlot: Slot
): FcResult[void] =
? self.update_time(dag, wallSlot)
@ -185,16 +184,16 @@ proc on_attestation*(
if beacon_block_root == Eth2Digest():
return ok()
if slot < self.checkpoints.time:
if attestation_slot < self.checkpoints.time:
for validator_index in attesting_indices:
# attestation_slot and target epoch must match, per attestation rules
self.backend.process_attestation(
validator_index, beacon_block_root, target.epoch)
validator_index, beacon_block_root, attestation_slot.epoch)
else:
self.queued_attestations.add(QueuedAttestation(
slot: slot,
attesting_indices: @attesting_indices,
block_root: beacon_block_root,
target_epoch: target.epoch))
self.queuedAttestations.add(QueuedAttestation(
slot: attestation_slot,
attesting_indices: toSeq(attesting_indices),
block_root: beacon_block_root))
ok()
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.1/specs/phase0/fork-choice.md#should_update_justified_checkpoint

View File

@ -9,7 +9,7 @@
import
# Standard library
std/[tables, options],
std/[options, tables],
# Status
stew/results,

View File

@ -190,44 +190,38 @@ func get_previous_epoch*(state: BeaconState): Epoch =
get_previous_epoch(get_current_epoch(state))
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/beacon-chain.md#compute_committee
func compute_committee_slice*(
active_validators, index, count: uint64): Slice[int] =
doAssert active_validators <= ValidatorIndex.high.uint64
let
start = (active_validators * index) div count
endIdx = (active_validators * (index + 1)) div count
start.int..(endIdx.int - 1)
func compute_committee*(shuffled_indices: seq[ValidatorIndex],
index: uint64, count: uint64): seq[ValidatorIndex] =
## Return the committee corresponding to ``indices``, ``seed``, ``index``,
## and committee ``count``.
## In this version, we pass in the shuffled indices meaning we no longer need
## the seed.
let
active_validators = shuffled_indices.len.uint64
start = (active_validators * index) div count
endIdx = (active_validators * (index + 1)) div count
# These assertions from compute_shuffled_index(...)
doAssert endIdx <= active_validators
doAssert active_validators <= 2'u64^40
slice = compute_committee_slice(shuffled_indices.lenu64, index, count)
# In spec, this calls get_shuffled_index() every time, but that's wasteful
# Here, get_beacon_committee() gets the shuffled version.
shuffled_indices[start.int .. (endIdx.int-1)]
shuffled_indices[slice]
func compute_committee_len*(active_validators: uint64,
index: uint64, count: uint64): uint64 =
func compute_committee_len*(
active_validators, index, count: uint64): uint64 =
## Return the committee corresponding to ``indices``, ``seed``, ``index``,
## and committee ``count``.
# indices only used here for its length, or for the shuffled version,
# so unlike spec, pass the shuffled version in directly.
let
start = (active_validators * index) div count
endIdx = (active_validators * (index + 1)) div count
slice = compute_committee_slice(active_validators, index, count)
# These assertions from compute_shuffled_index(...)
doAssert endIdx <= active_validators
doAssert active_validators <= 2'u64^40
# In spec, this calls get_shuffled_index() every time, but that's wasteful
# Here, get_beacon_committee() gets the shuffled version.
endIdx - start
(slice.b - slice.a + 1).uint64
# https://github.com/ethereum/eth2.0-specs/blob/v0.12.2/specs/phase0/beacon-chain.md#get_beacon_committee
func get_beacon_committee*(

View File

@ -4,7 +4,7 @@ import
chronos,
spec/datatypes
from times import Time, getTime, fromUnix, `<`, `-`
from times import Time, getTime, fromUnix, `<`, `-`, inNanoseconds
type
BeaconClock* = object
@ -25,7 +25,7 @@ type
# https://ethresear.ch/t/network-adjusted-timestamps/4187
genesis: Time
BeaconTime* = distinct int64 ## Seconds from beacon genesis time
BeaconTime* = distinct Duration ## Nanoseconds from beacon genesis time
proc init*(T: type BeaconClock, genesis_time: uint64): T =
let
@ -43,13 +43,22 @@ proc init*(T: type BeaconClock, state: BeaconState): T =
BeaconClock.init(state.genesis_time)
template `<`*(a, b: BeaconTime): bool =
int64(a) < int64(b)
Duration(a) < Duration(b)
template `<=`*(a, b: BeaconTime): bool =
int64(a) <= int64(b)
Duration(a) <= Duration(b)
template `+`*(t: BeaconTime, offset: Duration): BeaconTime =
BeaconTime(Duration(t) + offset)
template `-`*(t: BeaconTime, offset: Duration): BeaconTime =
BeaconTime(Duration(t) - offset)
template `-`*(a, b: BeaconTime): Duration =
Duration(a) - Duration(b)
func toSlot*(t: BeaconTime): tuple[afterGenesis: bool, slot: Slot] =
let ti = t.int64
let ti = seconds(Duration(t))
if ti >= 0:
(true, Slot(uint64(ti) div SECONDS_PER_SLOT))
else:
@ -61,13 +70,13 @@ func slotOrZero*(time: BeaconTime): Slot =
else: Slot(0)
func toBeaconTime*(c: BeaconClock, t: Time): BeaconTime =
BeaconTime(times.inSeconds(t - c.genesis))
BeaconTime(nanoseconds(inNanoseconds(t - c.genesis)))
func toSlot*(c: BeaconClock, t: Time): tuple[afterGenesis: bool, slot: Slot] =
c.toBeaconTime(t).toSlot()
func toBeaconTime*(s: Slot, offset = chronos.seconds(0)): BeaconTime =
BeaconTime(int64(uint64(s) * SECONDS_PER_SLOT) + seconds(offset))
func toBeaconTime*(s: Slot, offset = Duration()): BeaconTime =
BeaconTime(seconds(int64(uint64(s) * SECONDS_PER_SLOT)) + offset)
proc now*(c: BeaconClock): BeaconTime =
## Current time, in slots - this may end up being less than GENESIS_SLOT(!)
@ -76,10 +85,10 @@ proc now*(c: BeaconClock): BeaconTime =
proc fromNow*(c: BeaconClock, t: BeaconTime): tuple[inFuture: bool, offset: Duration] =
let now = c.now()
if int64(t) > int64(now):
(true, seconds(int64(t) - int64(now)))
if t > now:
(true, t - now)
else:
(false, seconds(int64(now) - int64(t)))
(false, now - t)
proc fromNow*(c: BeaconClock, slot: Slot): tuple[inFuture: bool, offset: Duration] =
c.fromNow(slot.toBeaconTime())
@ -115,8 +124,5 @@ func shortLog*(d: Duration): string =
tmp &= $frac & "m"
tmp
func `$`*(v: BeaconTime): string = $(int64(v))
func shortLog*(v: BeaconTime): int64 = v.int64
func `-`*(a, b: BeaconTime): Duration =
seconds(int64(a)) - seconds(int64(b))
func `$`*(v: BeaconTime): string = $Duration(v)
func shortLog*(v: BeaconTime): Duration = Duration(v)

View File

@ -93,7 +93,7 @@ cli do(slots = SLOTS_PER_EPOCH * 6,
data: data,
aggregation_bits: aggregation_bits,
signature: sig
), data.slot)
), [validatorIdx].toHashSet(), data.slot)
proc proposeBlock(slot: Slot) =
if rand(r, 1.0) > blockRatio:

View File

@ -8,13 +8,14 @@
{.used.}
import
std/[unittest, options],
chronicles,
std/unittest,
chronicles, chronos,
stew/byteutils,
./testutil, ./testblockutil,
../beacon_chain/spec/[crypto, datatypes, digest, validator, state_transition,
helpers, beaconstate, presets],
../beacon_chain/[beacon_node_types, attestation_pool, extras],
helpers, beaconstate, presets, network],
../beacon_chain/[
beacon_node_types, attestation_pool, attestation_aggregation, extras, time],
../beacon_chain/fork_choice/[fork_choice_types, fork_choice],
../beacon_chain/block_pools/[chain_dag, clearance]
@ -54,10 +55,10 @@ suiteReport "Attestation pool processing" & preset():
setup:
# Genesis state that results in 3 members per committee
var
chainDag = newClone(init(ChainDAGRef, defaultRuntimePreset, makeTestDB(SLOTS_PER_EPOCH * 3)))
quarantine = newClone(QuarantineRef())
chainDag = init(ChainDAGRef, defaultRuntimePreset, makeTestDB(SLOTS_PER_EPOCH * 3))
quarantine = QuarantineRef()
pool = newClone(AttestationPool.init(chainDag, quarantine))
state = newClone(loadTailState(chainDag))
state = newClone(chainDag.headState)
# Slot 0 is a finalized slot - won't be making attestations for it..
check:
process_slots(state.data, state.data.data.slot + 1)
@ -71,7 +72,8 @@ suiteReport "Attestation pool processing" & preset():
attestation = makeAttestation(
state.data.data, state.blck.root, beacon_committee[0], cache)
pool[].addAttestation(attestation, attestation.data.slot)
pool[].addAttestation(
attestation, [beacon_committee[0]].toHashSet(), attestation.data.slot)
check:
process_slots(state.data, MIN_ATTESTATION_INCLUSION_DELAY.Slot + 1)
@ -100,8 +102,10 @@ suiteReport "Attestation pool processing" & preset():
state.data.data, state.blck.root, bc1[0], cache)
# test reverse order
pool[].addAttestation(attestation1, attestation1.data.slot)
pool[].addAttestation(attestation0, attestation1.data.slot)
pool[].addAttestation(
attestation1, [bc1[0]].toHashSet, attestation1.data.slot)
pool[].addAttestation(
attestation0, [bc0[0]].toHashSet, attestation1.data.slot)
discard process_slots(state.data, MIN_ATTESTATION_INCLUSION_DELAY.Slot + 1)
@ -121,8 +125,10 @@ suiteReport "Attestation pool processing" & preset():
attestation1 = makeAttestation(
state.data.data, state.blck.root, bc0[1], cache)
pool[].addAttestation(attestation0, attestation0.data.slot)
pool[].addAttestation(attestation1, attestation1.data.slot)
pool[].addAttestation(
attestation0, [bc0[0]].toHashSet, attestation0.data.slot)
pool[].addAttestation(
attestation1, [bc0[1]].toHashSet, attestation1.data.slot)
check:
process_slots(state.data, MIN_ATTESTATION_INCLUSION_DELAY.Slot + 1)
@ -146,8 +152,10 @@ suiteReport "Attestation pool processing" & preset():
attestation0.combine(attestation1, {})
pool[].addAttestation(attestation0, attestation0.data.slot)
pool[].addAttestation(attestation1, attestation1.data.slot)
pool[].addAttestation(
attestation0, [bc0[0]].toHashSet, attestation0.data.slot)
pool[].addAttestation(
attestation1, [bc0[1]].toHashSet, attestation1.data.slot)
check:
process_slots(state.data, MIN_ATTESTATION_INCLUSION_DELAY.Slot + 1)
@ -170,8 +178,10 @@ suiteReport "Attestation pool processing" & preset():
attestation0.combine(attestation1, {})
pool[].addAttestation(attestation1, attestation1.data.slot)
pool[].addAttestation(attestation0, attestation0.data.slot)
pool[].addAttestation(
attestation1, [bc0[1]].toHashSet, attestation1.data.slot)
pool[].addAttestation(
attestation0, [bc0[0]].toHashSet, attestation0.data.slot)
check:
process_slots(state.data, MIN_ATTESTATION_INCLUSION_DELAY.Slot + 1)
@ -238,7 +248,8 @@ suiteReport "Attestation pool processing" & preset():
state.data.data, state.data.data.slot - 1, 1.CommitteeIndex, cache)
attestation0 = makeAttestation(state.data.data, b10.root, bc1[0], cache)
pool[].addAttestation(attestation0, attestation0.data.slot)
pool[].addAttestation(
attestation0, [bc1[0]].toHashSet, attestation0.data.slot)
let head2 = pool[].selectHead(b10Add[].slot)
@ -249,7 +260,8 @@ suiteReport "Attestation pool processing" & preset():
let
attestation1 = makeAttestation(state.data.data, b11.root, bc1[1], cache)
attestation2 = makeAttestation(state.data.data, b11.root, bc1[2], cache)
pool[].addAttestation(attestation1, attestation1.data.slot)
pool[].addAttestation(
attestation1, [bc1[1]].toHashSet, attestation1.data.slot)
let head3 = pool[].selectHead(b10Add[].slot)
let bigger = if b11.root.data < b10.root.data: b10Add else: b11Add
@ -258,7 +270,8 @@ suiteReport "Attestation pool processing" & preset():
# Ties broken lexicographically in spec -> ?
head3 == bigger[]
pool[].addAttestation(attestation2, attestation2.data.slot)
pool[].addAttestation(
attestation2, [bc1[2]].toHashSet, attestation2.data.slot)
let head4 = pool[].selectHead(b11Add[].slot)
@ -298,8 +311,8 @@ suiteReport "Attestation pool processing" & preset():
chainDag.updateFlags.incl {skipBLSValidation}
var cache = StateCache()
let
b10 = newClone(makeTestBlock(state.data, chainDag.tail.root, cache))
b10Add = chainDag.addRawBlock(quarantine, b10[]) do (
b10 = addTestBlock(state.data, chainDag.tail.root, cache)
b10Add = chainDag.addRawBlock(quarantine, b10) do (
blckRef: BlockRef, signedBlock: SignedBeaconBlock,
epochRef: EpochRef, state: HashedBeaconState):
# Callback add to fork choice if valid
@ -309,9 +322,6 @@ suiteReport "Attestation pool processing" & preset():
doAssert: head == b10Add[]
let block_ok = state_transition(defaultRuntimePreset, state.data, b10[], {}, noRollback)
doAssert: block_ok
# -------------------------------------------------------------
let b10_clone = b10 # Assumes deep copy
@ -326,14 +336,11 @@ suiteReport "Attestation pool processing" & preset():
let committees_per_slot =
get_committee_count_per_slot(state.data.data, Epoch epoch, cache)
for slot in start_slot ..< start_slot + SLOTS_PER_EPOCH:
let new_block = newClone(makeTestBlock(
state.data, block_root, cache, attestations = attestations))
let block_ok = state_transition(
defaultRuntimePreset, state.data, new_block[], {skipBLSValidation}, noRollback)
doAssert: block_ok
let new_block = addTestBlock(
state.data, block_root, cache, attestations = attestations)
block_root = new_block.root
let blockRef = chainDag.addRawBlock(quarantine, new_block[]) do (
let blockRef = chainDag.addRawBlock(quarantine, new_block) do (
blckRef: BlockRef, signedBlock: SignedBeaconBlock,
epochRef: EpochRef, state: HashedBeaconState):
# Callback add to fork choice if valid
@ -371,13 +378,76 @@ suiteReport "Attestation pool processing" & preset():
doAssert: chainDag.finalizedHead.slot != 0
pool[].prune()
doAssert: b10[].root notin pool.forkChoice.backend
doAssert: b10.root notin pool.forkChoice.backend
# Add back the old block to ensure we have a duplicate error
let b10Add_clone = chainDag.addRawBlock(quarantine, b10_clone[]) do (
let b10Add_clone = chainDag.addRawBlock(quarantine, b10_clone) do (
blckRef: BlockRef, signedBlock: SignedBeaconBlock,
epochRef: EpochRef, state: HashedBeaconState):
# Callback add to fork choice if valid
pool[].addForkChoice(epochRef, blckRef, signedBlock.message, blckRef.slot)
doAssert: b10Add_clone.error == Duplicate
suiteReport "Attestation validation " & preset():
setup:
# Genesis state that results in 3 members per committee
var
chainDag = init(ChainDAGRef, defaultRuntimePreset, makeTestDB(SLOTS_PER_EPOCH * 3))
quarantine = QuarantineRef()
pool = newClone(AttestationPool.init(chainDag, quarantine))
state = newClone(loadTailState(chainDag))
# Slot 0 is a finalized slot - won't be making attestations for it..
check:
process_slots(state.data, state.data.data.slot + 1)
wrappedTimedTest "Validation sanity":
chainDag.updateFlags.incl {skipBLSValidation}
var
cache: StateCache
for blck in makeTestBlocks(
chainDag.headState.data, chainDag.head.root, cache,
int(SLOTS_PER_EPOCH * 5), false):
let added = chainDag.addRawBlock(quarantine, blck) do (
blckRef: BlockRef, signedBlock: SignedBeaconBlock,
epochRef: EpochRef, state: HashedBeaconState):
# Callback add to fork choice if valid
pool[].addForkChoice(epochRef, blckRef, signedBlock.message, blckRef.slot)
check: added.isOk()
chainDag.updateHead(added[])
var
# Create an attestation for slot 1!
beacon_committee = get_beacon_committee(
chainDag.headState.data.data, chainDag.head.slot, 0.CommitteeIndex, cache)
attestation = makeAttestation(
chainDag.headState.data.data, chainDag.head.root, beacon_committee[0], cache)
committees_per_slot =
get_committee_count_per_slot(chainDag.headState.data.data,
attestation.data.slot.epoch, cache)
subnet = compute_subnet_for_attestation(
committees_per_slot,
attestation.data.slot, attestation.data.index.CommitteeIndex)
beaconTime = attestation.data.slot.toBeaconTime()
check:
validateAttestation(pool[], attestation, beaconTime, subnet).isOk
# Wrong subnet
validateAttestation(pool[], attestation, beaconTime, subnet + 1).isErr
# Too far in the future
validateAttestation(
pool[], attestation, beaconTime - 1.seconds, subnet + 1).isErr
# Too far in the past
validateAttestation(
pool[], attestation,
beaconTime - (SECONDS_PER_SLOT * SLOTS_PER_EPOCH - 1).int.seconds,
subnet + 1).isErr

View File

@ -118,7 +118,7 @@ suiteReport "Block pool processing" & preset():
db = makeTestDB(SLOTS_PER_EPOCH)
dag = init(ChainDAGRef, defaultRuntimePreset, db)
quarantine = QuarantineRef()
stateData = newClone(dag.loadTailState())
stateData = newClone(dag.headState)
cache = StateCache()
b1 = addTestBlock(stateData.data, dag.tail.root, cache)
b1Root = hash_tree_root(b1.message)
@ -330,22 +330,23 @@ suiteReport "chain DAG finalization tests" & preset():
process_slots(
tmpState[], tmpState.data.slot + (5 * SLOTS_PER_EPOCH).uint64)
let lateBlock = makeTestBlock(tmpState[], dag.head.root, cache)
let lateBlock = addTestBlock(tmpState[], dag.head.root, cache)
block:
let status = dag.addRawBlock(quarantine, blck, nil)
check: status.isOk()
assign(tmpState[], dag.headState.data)
for i in 0 ..< (SLOTS_PER_EPOCH * 6):
if i == 1:
# There are 2 heads now because of the fork at slot 1
check:
dag.heads.len == 2
blck = makeTestBlock(
dag.headState.data, dag.head.root, cache,
blck = addTestBlock(
tmpState[], dag.head.root, cache,
attestations = makeFullAttestations(
dag.headState.data.data, dag.head.root,
dag.headState.data.data.slot, cache, {}))
tmpState[].data, dag.head.root, tmpState[].data.slot, cache, {}))
let added = dag.addRawBlock(quarantine, blck, nil)
check: added.isOk()
dag.updateHead(added[])
@ -436,15 +437,10 @@ suiteReport "chain DAG finalization tests" & preset():
quarantine = QuarantineRef()
cache = StateCache()
wrappedTimedTest "init with gaps" & preset():
for i in 0 ..< (SLOTS_PER_EPOCH * 6 - 2):
var
blck = makeTestBlock(
dag.headState.data, dag.head.root, cache,
attestations = makeFullAttestations(
dag.headState.data.data, dag.head.root,
dag.headState.data.data.slot, cache, {}))
timedTest "init with gaps" & preset():
for blck in makeTestBlocks(
dag.headState.data, dag.head.root, cache, int(SLOTS_PER_EPOCH * 6 - 2),
true):
let added = dag.addRawBlock(quarantine, blck, nil)
check: added.isOk()
dag.updateHead(added[])

View File

@ -248,3 +248,27 @@ proc makeFullAttestations*(
attestation.signature = agg.finish()
result.add attestation
iterator makeTestBlocks*(
state: HashedBeaconState,
parent_root: Eth2Digest,
cache: var StateCache,
blocks: int,
attested: bool,
flags: set[UpdateFlag] = {}
): SignedBeaconBlock =
var
state = assignClone(state)
parent_root = parent_root
for _ in 0..<blocks:
let attestations = if attested:
makeFullAttestations(
state[].data, parent_root,
state[].data.slot, cache, flags)
else:
@[]
let blck = addTestBlock(
state[], parent_root, cache, attestations = attestations, flags = flags)
yield blck
parent_root = blck.root