approvals

package
v0.33.17 Latest Latest
Warning

This package is not in the latest version of its module.

Go to latest
Published: Apr 17, 2024 License: AGPL-3.0 Imports: 33 Imported by: 2

README

Tracking result approvals

Context

Among other things, an ExecutionResult specifies

  • BlockID: the ID of the executed block
  • PreviousResultID: the ID of the result that was used as starting state to run the computation

This implies the ExecutionResults form a tree. We call this the Execution Tree. However, nodes do not need to retain the entire tree, but can prune levels of lower height. By pruning lower levels, the full tree decomposes into a forest of subtrees.

Multiple verifier assignments

For a single Execution Result, there can be multiple assignments of Verification Nodes to the individual chunks. This is because:

  • Execution results are incorporated in blocks.
  • The block that incorporates the result is used to determine the verifier assignment.
  • The same result r can be incorporated in different forks.
  • The different blocks incorporating r will have different Sources of Randomness. Hence, the verifier assignments differ for the different forks.

At some point, verifier assignments can be discarded. For example, if the block incorporated the result is orphaned.

Assignment Collectors

From a computational perspective, it makes sense to group all assignments for the same result:

  • A ResultApproval is for a specific execution result. There is nothing in the approval that ties it to a specific assignment. So theoretically, the same approval could count towards multiple different assignments.
  • The dominant computational cost when processing a ResultApproval is verifying its correctness (cryptographic signature and SPoCK proof). In contrast, determining which assignments the approval can be counted towards is negligible. In other words, tracking more than the minimally necessary assignments adds no noteworthy computational overhead)

Hence, it makes sense to define the AssignmentCollector. It processes the ResultApprovals for one particular execution result:

  • Specifically, it memorizes all known verifier assignments for the result.
  • When it receives a ResultApproval, it verifies the approval once and then adds the approval to eligible assignments.
Assignment Collector Tree

It is beneficial to arrange the Assignment Collectors as a tree (or a forest depending on pruning). This allows us to draw conclusions

As illustrated by the figure above, the AssignmentCollectors form a tree Formally, we define:

  • All Verification assignments for the same result from an equivalence class. The equivalence class represents one vertex in the AssignmentCollectorTree. It is implemented as AssignmentCollector.

  • The edges are placed exactly like in the Execution Tree:

    • Let r[B] be an execution result for block B. We denote the corresponding AssignmentCollector as c[r[B]].
    • Let r_parent := r[B].PreviousResultID be the parent result and c[r_parent] the corresponding AssignmentCollector.

    In the Execution Tree, there is an edge r[B] --> r_parent. Correspondingly, we place the edge c[r[B]] --> c[r_parent] in the Assignment Collector Tree.

  • Lastly, for an AssignmentCollector, we define a level as the height of the executed block.

Assignment Collector Tree

Orphaning forks in the Assignment Collector Tree

There are two scenarios when we can orphan entire forks of the Assignment Collector Tree:

  1. The executed blocks were orphaned. This happens as a consequence of block finalization by the local HotStuff instance.
  2. If an execution result conflicts with a finalized result seal, the result itself as well as all the derived results are orphaned.

In addition, we can prune all levels in the Assignment Collector Tree that are below the latest finalized seal.

When an AssignmentCollector is orphaned, we can stop processing the corresponding approvals. However, for the following reason, we should not right away discard the respective vertices in the Assignment Collector Tree:

  • We might later learn about derives results (e.g. results for child blocks or different execution results).
  • In order to make the determination whether a new AssignmentCollector should be orphaned, we retain the orphaned forks in the Assignment Collector Tree. When adding an AssignmentCollector that extends an already orphaned fork, it is also orphaned.

Orphaned forks will eventually be pruned by height, when sealing processes beyond the fork's height.

The Assignment Collector's States

We already have discussed two different modes an assignment collector could operate in:

  • VerifyingApprovals: it is processing incoming ResultApprovals
  • Orphaned: it has been orphaned and discards all ResultApprovals

The Assignment Collector Tree is populated in a fully concurrent manner. Hence, when adding an AssignmentCollector, it can happen that the parent is not yet present (see following figure for illustration). This is the mode:

  • CachingApprovals: the collector caches all approvals without processing them. This mode is relatively rare and largely occurs as an edge-case if the node is behind and catching up (i.e. processing many blocks in a relatively short time). If the node is behind, it is likely that the result is already sealed by the time it caught up. We permit a simple implementation that is to not keep track of the order in which it received the approvals.

Assignment Collector Tree

In the future, we will also add further states for 'Extensive Checking Mode':

  • ExtensiveCheckingMode: the AssignmentCollector has not received sufficient approvals and sealing is lagging behind (exceeding a certain threshold). In this case, the should trigger extensive checkig mode, i.e. request additional verifiers to check.

Implementation Comments

Compare And Repeat Pattern

In multiple places we employ the Compare And Repeat Pattern. This pattern is applicable when we have:

  • an atomically-updatable state value
  • some business logic that requires an up-to-date value of the state

With the Compare And Repeat Pattern, we guarantee that the business logic was executed with the most recent state.

Pattern Details
  1. Atomically read the state before the operation.
  2. Execute the operation on the retrieved state.
  3. Atomically read the state after the operation.
    • If the state changed, we updated a stale state. In this case we go back to step 1.
    • If the state remained unchanged, we updated the most recent state. Hence, we are done.
Key for caching ResultApprovals

In case we don't process a ResultApproval right away, we generally cache it. If done naively, the caching could be exploited by byzantine nodes: they could send lots of approvals for the same chunk but vary some field in their approvals that is not validated right away (e.g. the SPoCK proof). If consensus nodes used the ID of the full approval as key for the cache, the malicious approvals would have all different keys, which would allow a malicious node to flood the cache.

Instead, we only want to cache one approval per Verification node for each specific chunk. Therefore, we use ResultApproval.PartialID(), which is only computed over the chunk-identifier plus the verifier's node ID.

Emergency Sealing

Emergency sealing is a temporary fallback to maintain liveness in case there are problems with verification. In a nutshell, emergency sealing means that (selected) consensus nodes can seal blocks without sufficient approvals from verification nodes.

Conceptually, emergency sealing follows a similar process as regular sealing: we consider verification assignments that have not yet been sealed (or orphaned). We inspect each assignment individually whether it satisfies the emergency sealing conditions (details below). If this is the case, we generate a candidate seal and place it in the consensus node's local IncorporatedResultSeals mempool. The block builder will then pick out a continuous sequence of seals from the mempool if such exists. This simplifies the sealing logic (happy path as well as emergency sealing fallback), because we can evaluate sealing conditions for each incorporated Execution Result [ER] individually.

Consider block A, whose Execution Result [ER] was incorporated in block B (illustrated below). The decision whether the ER can be emergency sealed is governed by two protocol parameters: DefaultEmergencySealingThresholdForFinalization and DefaultEmergencySealingThresholdForVerification. For an ER to be emergency sealed, all of the following conditions have to be satisfied:

Emergency sealing

  1. Let Δh1 be the height difference between the latest finalized block and block A. We require that Δh1 > DefaultEmergencySealingThresholdForFinalization.
    • This means that block A must have at least DefaultEmergencySealingThresholdForFinalization unsealed but finalized descendants.
    • This condition is enforced by the VerifyingAssignmentCollector.
    • We also use this condition in sealing.Core to evaluate which height range we need to check for emergency sealing.
  2. Let Δh2 be the height difference between the latest finalized block and block B, which incorporates the ER for block A. We require that Δh2 > DefaultEmergencySealingThresholdForVerification
    • This means that the block incorporating the ER must have at least DefaultEmergencySealingThresholdForVerification finalized descendants.
    • This condition is enforced by the VerifyingAssignmentCollector.
  3. There must be consistent execution results from at least two different Execution Nodes to even consider a result for emergency sealing.
    • This condition is enforced by the consensus.IncorporatedResultSeals. This mempool stores all candidate seals, but hides them from the block builder until at least two consistent execution results are known.
    • Comment: as a temporary safety measure, requiring at least two consistent results must hold for any block, no matter whether the seal was generated through the happy path or via emergency sealing.
  4. The parent block has to be sealed.
    • This condition is enforced by the consensus.Builder

Documentation

Index

Constants

View Source
const DefaultEmergencySealingThresholdForFinalization = 100

DefaultEmergencySealingThresholdForFinalization is the minimal number of unsealed but finalized descendants that a block must have in order to be eligible for emergency sealing (further conditions apply for emergency sealing).

View Source
const DefaultEmergencySealingThresholdForVerification = 25

DefaultEmergencySealingThresholdForVerification is the minimal number of finalized descendants that the block _incorporating_ an Execution Result [ER] must have for the ER to be eligible for emergency sealing (further conditions apply for emergency sealing).

Variables

View Source
var (
	ErrInvalidCollectorStateTransition = errors.New("invalid state transition")
	ErrDifferentCollectorState         = errors.New("different state")
)

Functions

This section is empty.

Types

type AggregatedSignatures

type AggregatedSignatures struct {
	// contains filtered or unexported fields
}

AggregatedSignatures is an utility struct that provides concurrency safe access to map of aggregated signatures indexed by chunk index

func NewAggregatedSignatures

func NewAggregatedSignatures(chunks uint64) (*AggregatedSignatures, error)

NewAggregatedSignatures instantiates a AggregatedSignatures. Requires that number of chunks is positive integer. Errors otherwise.

func (*AggregatedSignatures) ChunksWithoutAggregatedSignature added in v0.17.6

func (as *AggregatedSignatures) ChunksWithoutAggregatedSignature() []uint64

ChunksWithoutAggregatedSignature returns indexes of chunks that don't have an aggregated signature

func (*AggregatedSignatures) Collect

Collect returns array with aggregated signature for each chunk

func (*AggregatedSignatures) HasSignature

func (as *AggregatedSignatures) HasSignature(chunkIndex uint64) bool

HasSignature returns boolean depending if we have signature for particular chunk

func (*AggregatedSignatures) PutSignature

func (as *AggregatedSignatures) PutSignature(chunkIndex uint64, aggregatedSignature flow.AggregatedSignature) (uint64, error)

PutSignature adds the AggregatedSignature from the collector to `aggregatedSignatures`. The returned int is the resulting number of approved chunks. Errors if chunk index exceeds valid range.

type ApprovalCollector

type ApprovalCollector struct {
	// contains filtered or unexported fields
}

ApprovalCollector is responsible for distributing work to chunk collectorTree, collecting aggregated signatures for chunks that reached seal construction threshold, creating and submitting seal candidates once signatures for every chunk are aggregated.

func NewApprovalCollector

func NewApprovalCollector(
	log zerolog.Logger,
	result *flow.IncorporatedResult,
	incorporatedBlock *flow.Header,
	executedBlock *flow.Header,
	assignment *chunks.Assignment,
	seals mempool.IncorporatedResultSeals,
	requiredApprovalsForSealConstruction uint,
) (*ApprovalCollector, error)

func (*ApprovalCollector) CollectMissingVerifiers

func (c *ApprovalCollector) CollectMissingVerifiers() map[uint64]flow.IdentifierList

CollectMissingVerifiers collects ids of verifiers who haven't provided an approval for particular chunk Returns: map { ChunkIndex -> []VerifierId }

func (*ApprovalCollector) IncorporatedBlock

func (c *ApprovalCollector) IncorporatedBlock() *flow.Header

IncorporatedBlock returns the block which incorporates execution result

func (*ApprovalCollector) IncorporatedBlockID

func (c *ApprovalCollector) IncorporatedBlockID() flow.Identifier

IncorporatedBlockID returns the ID of block which incorporates execution result

func (*ApprovalCollector) IncorporatedResult added in v0.18.3

func (c *ApprovalCollector) IncorporatedResult() *flow.IncorporatedResult

IncorporatedResult returns the incorporated Result this ApprovalCollector is for

func (*ApprovalCollector) ProcessApproval

func (c *ApprovalCollector) ProcessApproval(approval *flow.ResultApproval) error

ProcessApproval performs processing of result approvals and bookkeeping of aggregated signatures for every chunk. Triggers sealing of execution result when processed last result approval needed for sealing. Returns: - engine.InvalidInputError - result approval is invalid - exception in case of any other error, usually this is not expected - nil on success

func (*ApprovalCollector) SealResult

func (c *ApprovalCollector) SealResult() error

type ApprovalsCache added in v0.20.0

type ApprovalsCache struct {
	// contains filtered or unexported fields
}

ApprovalsCache encapsulates a map for storing result approvals indexed by approval partial ID to provide concurrent access.

func NewApprovalsCache

func NewApprovalsCache(sizeHint uint) *ApprovalsCache

func (*ApprovalsCache) All added in v0.20.0

func (c *ApprovalsCache) All() []*flow.ResultApproval

All returns all stored approvals

func (*ApprovalsCache) Get added in v0.20.0

func (c *ApprovalsCache) Get(approvalID flow.Identifier) *flow.ResultApproval

Get returns ResultApproval for the given ID (or nil if none is stored)

func (*ApprovalsCache) Put added in v0.20.0

func (c *ApprovalsCache) Put(approvalID flow.Identifier, approval *flow.ResultApproval) bool

Put saves approval into cache; returns true iff approval was newly added the approvalID should be calculated as `approval.Body.PartialID()` it is taken as an input so that the caller can optimize by reusing the calculated approvalID.

type AssignmentCollector

type AssignmentCollector interface {
	AssignmentCollectorState

	// ChangeProcessingStatus changes the AssignmentCollector's internal processing
	// status. The operation is implemented as an atomic compare-and-swap, i.e. the
	// state transition is only executed if AssignmentCollector's internal state is
	// equal to `expectedValue`. The return indicates whether the state was updated.
	// The implementation only allows the transitions
	//         CachingApprovals -> VerifyingApprovals
	//    and                      VerifyingApprovals -> Orphaned
	// Error returns:
	// * nil if the state transition was successfully executed
	// * ErrDifferentCollectorState if the AssignmentCollector's state is different than expectedCurrentStatus
	// * ErrInvalidCollectorStateTransition if the given state transition is impossible
	// * all other errors are unexpected and potential symptoms of internal bugs or state corruption (fatal)
	ChangeProcessingStatus(expectedValue, newValue ProcessingStatus) error
}

AssignmentCollector tracks the known verifier assignments for a particular execution result. For the same result, there can be multiple different assignments. This happens if the result is incorporated in different forks, because in each fork a unique verifier assignment is generated using the incorporating block's unique 'source of randomness'. An AssignmentCollector can be in different states (enumerated by ProcessingStatus). State transitions are atomic and concurrency safe. The high-level AssignmentCollector implements the logic for state transitions, while it delegates the state-specific portion of the logic to `AssignmentCollectorState`.

type AssignmentCollectorBase added in v0.20.0

type AssignmentCollectorBase struct {
	// contains filtered or unexported fields
}

AssignmentCollectorBase holds the shared data and functionality for implementations of the AssignmentCollectorBase holds common dependencies and immutable values that are shared by the different states of an AssignmentCollector. It is indented as the base struct for the different `AssignmentCollectorState` implementations.

func NewAssignmentCollectorBase added in v0.20.0

func NewAssignmentCollectorBase(logger zerolog.Logger,
	workerPool *workerpool.WorkerPool,
	result *flow.ExecutionResult,
	state protocol.State,
	headers storage.Headers,
	assigner module.ChunkAssigner,
	seals mempool.IncorporatedResultSeals,
	sigHasher hash.Hasher,
	approvalConduit network.Conduit,
	requestTracker *RequestTracker,
	requiredApprovalsForSealConstruction uint,
) (AssignmentCollectorBase, error)

func (*AssignmentCollectorBase) Block added in v0.20.0

func (cb *AssignmentCollectorBase) Block() *flow.Header

func (*AssignmentCollectorBase) BlockID added in v0.20.0

func (cb *AssignmentCollectorBase) BlockID() flow.Identifier

func (*AssignmentCollectorBase) OnInvalidApproval added in v0.20.0

func (cb *AssignmentCollectorBase) OnInvalidApproval(approval *flow.ResultApproval, err error)

OnInvalidApproval logs in invalid approval

func (*AssignmentCollectorBase) Result added in v0.20.0

func (*AssignmentCollectorBase) ResultID added in v0.20.0

func (cb *AssignmentCollectorBase) ResultID() flow.Identifier

type AssignmentCollectorState added in v0.20.0

type AssignmentCollectorState interface {
	// BlockID returns the ID of the executed block.
	BlockID() flow.Identifier
	// Block returns the header of the executed block.
	Block() *flow.Header
	// ResultID returns the ID of the result this assignment collector tracks.
	ResultID() flow.Identifier
	// Result returns the result this assignment collector tracks.
	Result() *flow.ExecutionResult

	// ProcessIncorporatedResult starts tracking the approval for IncorporatedResult.
	// Method is idempotent.
	// Error Returns:
	//  * no errors expected during normal operation;
	//    errors might be symptoms of bugs or internal state corruption (fatal)
	ProcessIncorporatedResult(incorporatedResult *flow.IncorporatedResult) error

	// ProcessApproval ingests Result Approvals and triggers sealing of execution result
	// when sufficient approvals have arrived. Method is idempotent.
	// Error Returns:
	//  * nil in case of success (outdated approvals might be silently discarded)
	//  * engine.InvalidInputError if the result approval is invalid
	//  * any other errors might be symptoms of bugs or internal state corruption (fatal)
	ProcessApproval(approval *flow.ResultApproval) error

	// CheckEmergencySealing checks whether this AssignmentCollector can be emergency
	// sealed. If this is the case, the AssignmentCollector produces a candidate seal
	// as part of this method call. No errors are expected during normal operations.
	CheckEmergencySealing(observer consensus.SealingObservation, finalizedBlockHeight uint64) error

	// RequestMissingApprovals sends requests for missing approvals to the respective
	// verification nodes. Returns number of requests made. No errors are expected
	// during normal operations.
	RequestMissingApprovals(observer consensus.SealingObservation, maxHeightForRequesting uint64) (uint, error)

	// ProcessingStatus returns the AssignmentCollector's ProcessingStatus (state descriptor).
	ProcessingStatus() ProcessingStatus
}

AssignmentCollectorState represents an AssignmentCollector in one specific state (without any knowledge about state transitions).

func NewOrphanAssignmentCollector added in v0.20.0

func NewOrphanAssignmentCollector(collectorBase AssignmentCollectorBase) AssignmentCollectorState

type AssignmentCollectorStateMachine added in v0.20.0

type AssignmentCollectorStateMachine struct {
	AssignmentCollectorBase

	sync.Mutex
	// contains filtered or unexported fields
}

AssignmentCollectorStateMachine implements the `AssignmentCollector` interface. It wraps the current `AssignmentCollectorState` and provides logic for state transitions. Any state-specific logic is delegated to the state-specific instance. AssignmentCollectorStateMachine is fully concurrent.

Comment on concurrency safety for state-specific business logic:

  • AssignmentCollectorStateMachine processes state updates concurrently with state-specific business logic. Hence, it can happen that we update a stale state.
  • To guarantee that we hand inputs to the latest state, we employ a "Compare And Repeat Pattern": we atomically read the state before and after the operation. If the state changed, we updated a stale state. We repeat until we confirm that the latest state was updated.

func NewAssignmentCollectorStateMachine added in v0.20.0

func NewAssignmentCollectorStateMachine(collectorBase AssignmentCollectorBase) *AssignmentCollectorStateMachine

func (*AssignmentCollectorStateMachine) ChangeProcessingStatus added in v0.20.0

func (asm *AssignmentCollectorStateMachine) ChangeProcessingStatus(expectedCurrentStatus, newStatus ProcessingStatus) error

ChangeProcessingStatus changes the AssignmentCollector's internal processing status. The operation is implemented as an atomic compare-and-swap, i.e. the state transition is only executed if AssignmentCollector's internal state is equal to `expectedValue`. The return indicates whether the state was updated. The implementation only allows the following transitions:

CachingApprovals   -> VerifyingApprovals
CachingApprovals   -> Orphaned
VerifyingApprovals -> Orphaned

Error returns: * nil if the state transition was successfully executed * ErrDifferentCollectorState if the AssignmentCollector's state is different than expectedCurrentStatus * ErrInvalidCollectorStateTransition if the given state transition is impossible * all other errors are unexpected and potential symptoms of internal bugs or state corruption (fatal)

func (*AssignmentCollectorStateMachine) CheckEmergencySealing added in v0.20.0

func (asm *AssignmentCollectorStateMachine) CheckEmergencySealing(observer consensus.SealingObservation, finalizedBlockHeight uint64) error

CheckEmergencySealing checks whether this AssignmentCollector can be emergency sealed. If this is the case, the AssignmentCollector produces a candidate seal as part of this method call. No errors are expected during normal operations.

func (*AssignmentCollectorStateMachine) ProcessApproval added in v0.20.0

func (asm *AssignmentCollectorStateMachine) ProcessApproval(approval *flow.ResultApproval) error

ProcessApproval ingests Result Approvals and triggers sealing of execution result when sufficient approvals have arrived. Error Returns:

  • nil in case of success (outdated approvals might be silently discarded)
  • engine.InvalidInputError if the result approval is invalid
  • any other errors might be symptoms of bugs or internal state corruption (fatal)

func (*AssignmentCollectorStateMachine) ProcessIncorporatedResult added in v0.20.0

func (asm *AssignmentCollectorStateMachine) ProcessIncorporatedResult(incorporatedResult *flow.IncorporatedResult) error

ProcessIncorporatedResult starts tracking the approval for IncorporatedResult. Method is idempotent. Error Returns:

  • no errors expected during normal operation; errors might be symptoms of bugs or internal state corruption (fatal)

func (*AssignmentCollectorStateMachine) ProcessingStatus added in v0.20.0

func (asm *AssignmentCollectorStateMachine) ProcessingStatus() ProcessingStatus

ProcessingStatus returns the AssignmentCollector's ProcessingStatus (state descriptor).

func (*AssignmentCollectorStateMachine) RequestMissingApprovals added in v0.20.0

func (asm *AssignmentCollectorStateMachine) RequestMissingApprovals(observer consensus.SealingObservation, maxHeightForRequesting uint64) (uint, error)

RequestMissingApprovals sends requests for missing approvals to the respective verification nodes. Returns number of requests made. No errors are expected during normal operations.

type AssignmentCollectorTree

type AssignmentCollectorTree struct {
	// contains filtered or unexported fields
}

AssignmentCollectorTree is a mempool holding assignment collectors, which is aware of the tree structure formed by the execution results. The mempool supports pruning by height: only collectors descending from the latest sealed and finalized result are relevant. Safe for concurrent access. Internally, the mempool utilizes the LevelledForest.

func NewAssignmentCollectorTree

func NewAssignmentCollectorTree(lastSealed *flow.Header, headers storage.Headers, createCollector NewCollectorFactoryMethod) *AssignmentCollectorTree

func (*AssignmentCollectorTree) FinalizeForkAtLevel

func (t *AssignmentCollectorTree) FinalizeForkAtLevel(finalized *flow.Header, sealed *flow.Header) error

FinalizeForkAtLevel orphans forks in the AssignmentCollectorTree and prunes levels below the sealed finalized height. When a block is finalized we can mark results for conflicting forks as orphaned and stop processing approvals for them. Eventually all forks will be cleaned up by height.

func (*AssignmentCollectorTree) GetCollector

func (t *AssignmentCollectorTree) GetCollector(resultID flow.Identifier) AssignmentCollector

GetCollector returns assignment collector for the given result.

func (*AssignmentCollectorTree) GetCollectorsByInterval

func (t *AssignmentCollectorTree) GetCollectorsByInterval(from, to uint64) []AssignmentCollector

GetCollectorsByInterval returns all collectors in state `VerifyingApprovals` whose executed block has height in [from; to)

func (*AssignmentCollectorTree) GetOrCreateCollector

func (t *AssignmentCollectorTree) GetOrCreateCollector(result *flow.ExecutionResult) (*LazyInitCollector, error)

GetOrCreateCollector performs lazy initialization of AssignmentCollector using double-checked locking.

func (*AssignmentCollectorTree) GetSize added in v0.17.6

func (t *AssignmentCollectorTree) GetSize() uint64

type BaseApprovalsTestSuite

type BaseApprovalsTestSuite struct {
	suite.Suite

	ParentBlock         *flow.Header    // parent of sealing candidate
	Block               *flow.Header    // candidate for sealing
	IncorporatedBlock   *flow.Header    // block that incorporated result
	VerID               flow.Identifier // for convenience, node id of first verifier
	Chunks              flow.ChunkList  // list of chunks of execution result
	ChunksAssignment    *chunks.Assignment
	AuthorizedVerifiers map[flow.Identifier]*flow.Identity // map of authorized verifier identities for execution result
	PublicKey           *module.PublicKey                  // public key used to mock signature verifications
	SigHasher           hash.Hasher                        // used to verify signatures
	IncorporatedResult  *flow.IncorporatedResult
}

BaseApprovalsTestSuite is a base suite for testing approvals processing related functionality At nutshell generates mock data that can be used to create approvals and provides all needed data to validate those approvals for respected execution result.

func (*BaseApprovalsTestSuite) SetupTest

func (s *BaseApprovalsTestSuite) SetupTest()

type BaseAssignmentCollectorTestSuite added in v0.20.0

type BaseAssignmentCollectorTestSuite struct {
	BaseApprovalsTestSuite

	WorkerPool        *workerpool.WorkerPool
	Blocks            map[flow.Identifier]*flow.Header
	State             *protocol.State
	Snapshots         map[flow.Identifier]*protocol.Snapshot
	Headers           *storage.Headers
	Assigner          *module.ChunkAssigner
	SealsPL           *mempool.IncorporatedResultSeals
	Conduit           *mocknetwork.Conduit
	FinalizedAtHeight map[uint64]*flow.Header
	IdentitiesCache   map[flow.Identifier]map[flow.Identifier]*flow.Identity // helper map to store identities for given block
	RequestTracker    *RequestTracker
}

BaseAssignmentCollectorTestSuite is a base suite for testing assignment collectors, contains mocks for all classes that are used in base assignment collector and can be reused in different test suites.

func (*BaseAssignmentCollectorTestSuite) MarkFinalized added in v0.21.0

func (s *BaseAssignmentCollectorTestSuite) MarkFinalized(block *flow.Header)

func (*BaseAssignmentCollectorTestSuite) SetupTest added in v0.20.0

func (s *BaseAssignmentCollectorTestSuite) SetupTest()

func (*BaseAssignmentCollectorTestSuite) TearDownTest added in v0.21.0

func (s *BaseAssignmentCollectorTestSuite) TearDownTest()

type CachingAssignmentCollector added in v0.20.0

type CachingAssignmentCollector struct {
	AssignmentCollectorBase
	// contains filtered or unexported fields
}

CachingAssignmentCollector is an AssignmentCollectorState with the fixed `ProcessingStatus` of `CachingApprovals`.

func NewCachingAssignmentCollector added in v0.20.0

func NewCachingAssignmentCollector(collectorBase AssignmentCollectorBase) *CachingAssignmentCollector

func (*CachingAssignmentCollector) CheckEmergencySealing added in v0.20.0

func (*CachingAssignmentCollector) GetApprovals added in v0.20.0

func (ac *CachingAssignmentCollector) GetApprovals() []*flow.ResultApproval

func (*CachingAssignmentCollector) GetIncorporatedResults added in v0.20.0

func (ac *CachingAssignmentCollector) GetIncorporatedResults() []*flow.IncorporatedResult

func (*CachingAssignmentCollector) ProcessApproval added in v0.20.0

func (ac *CachingAssignmentCollector) ProcessApproval(approval *flow.ResultApproval) error

ProcessApproval ingests Result Approvals and triggers sealing of execution result when sufficient approvals have arrived. Error Returns:

  • nil in case of success (outdated approvals might be silently discarded)
  • engine.InvalidInputError if the result approval is invalid
  • any other errors might be symptoms of bugs or internal state corruption (fatal)

func (*CachingAssignmentCollector) ProcessIncorporatedResult added in v0.20.0

func (ac *CachingAssignmentCollector) ProcessIncorporatedResult(incorporatedResult *flow.IncorporatedResult) error

ProcessIncorporatedResult starts tracking the approval for IncorporatedResult. Method is idempotent. Error Returns:

  • no errors expected during normal operation; errors might be symptoms of bugs or internal state corruption (fatal)

func (*CachingAssignmentCollector) ProcessingStatus added in v0.20.0

func (ac *CachingAssignmentCollector) ProcessingStatus() ProcessingStatus

func (*CachingAssignmentCollector) RequestMissingApprovals added in v0.20.0

func (ac *CachingAssignmentCollector) RequestMissingApprovals(consensus.SealingObservation, uint64) (uint, error)

type ChunkApprovalCollector

type ChunkApprovalCollector struct {
	// contains filtered or unexported fields
}

ChunkApprovalCollector implements logic for checking chunks against assignments as well as accumulating signatures of already checked approvals.

func NewChunkApprovalCollector

func NewChunkApprovalCollector(assignment map[flow.Identifier]struct{}, requiredApprovalsForSealConstruction uint) *ChunkApprovalCollector

func (*ChunkApprovalCollector) GetMissingSigners

func (c *ChunkApprovalCollector) GetMissingSigners() flow.IdentifierList

GetMissingSigners returns ids of approvers that are present in assignment but didn't provide approvals

func (*ChunkApprovalCollector) ProcessApproval

func (c *ChunkApprovalCollector) ProcessApproval(approval *flow.ResultApproval) (flow.AggregatedSignature, bool)

ProcessApproval performs processing and bookkeeping of single approval

type IncorporatedResultsCache added in v0.20.0

type IncorporatedResultsCache struct {
	// contains filtered or unexported fields
}

IncorporatedResultsCache encapsulates a map for storing IncorporatedResults to provide concurrent access.

func NewIncorporatedResultsCache added in v0.20.0

func NewIncorporatedResultsCache(sizeHint uint) *IncorporatedResultsCache

func (*IncorporatedResultsCache) All added in v0.20.0

All returns all stored approvals

func (*IncorporatedResultsCache) Get added in v0.20.0

Get returns IncorporatedResult for the given ID (or nil if none is stored)

func (*IncorporatedResultsCache) Put added in v0.20.0

Put saves IncorporatedResult into cache; returns true iff IncorporatedResult was newly added

type LazyInitCollector

type LazyInitCollector struct {
	Collector AssignmentCollector
	Created   bool // whether collector was created or retrieved from cache
}

LazyInitCollector is a helper structure that is used to return collector which is lazy initialized

type LruCache

type LruCache struct {
	// contains filtered or unexported fields
}

LruCache is a wrapper over `simplelru.LRUCache` that provides needed api for processing result approvals Extends functionality of `simplelru.LRUCache` by introducing additional index for quicker access.

func NewApprovalsLRUCache

func NewApprovalsLRUCache(limit uint) *LruCache

func (*LruCache) Get

func (c *LruCache) Get(approvalID flow.Identifier) *flow.ResultApproval

func (*LruCache) Peek

func (c *LruCache) Peek(approvalID flow.Identifier) *flow.ResultApproval

func (*LruCache) Put

func (c *LruCache) Put(approval *flow.ResultApproval)

func (*LruCache) TakeByResultID

func (c *LruCache) TakeByResultID(resultID flow.Identifier) []*flow.ResultApproval

type NewCollectorFactoryMethod

type NewCollectorFactoryMethod = func(result *flow.ExecutionResult) (AssignmentCollector, error)

NewCollectorFactoryMethod is a factory method to generate an AssignmentCollector for an execution result

type OrphanAssignmentCollector added in v0.20.0

type OrphanAssignmentCollector struct {
	AssignmentCollectorBase
}

OrphanAssignmentCollector is an AssignmentCollectorState with the fixed `ProcessingStatus` of `Orphaned`.

func (*OrphanAssignmentCollector) CheckEmergencySealing added in v0.20.0

func (*OrphanAssignmentCollector) ProcessApproval added in v0.20.0

func (oc *OrphanAssignmentCollector) ProcessApproval(*flow.ResultApproval) error

func (*OrphanAssignmentCollector) ProcessIncorporatedResult added in v0.20.0

func (oc *OrphanAssignmentCollector) ProcessIncorporatedResult(*flow.IncorporatedResult) error

func (*OrphanAssignmentCollector) ProcessingStatus added in v0.20.0

func (oc *OrphanAssignmentCollector) ProcessingStatus() ProcessingStatus

func (*OrphanAssignmentCollector) RequestMissingApprovals added in v0.20.0

func (oc *OrphanAssignmentCollector) RequestMissingApprovals(consensus.SealingObservation, uint64) (uint, error)

type ProcessingStatus added in v0.20.0

type ProcessingStatus int

ProcessingStatus is a state descriptor for the AssignmentCollector.

const (
	// CachingApprovals is a state descriptor for the AssignmentCollector. In this state,
	// the collector is currently caching approvals but _not_ yet processing them.
	CachingApprovals ProcessingStatus = iota
	// VerifyingApprovals is a state descriptor for the AssignmentCollector. In this state,
	// the collector is processing approvals.
	VerifyingApprovals
	// Orphaned is a state descriptor for the AssignmentCollector. In this state,
	// the collector discards all approvals.
	Orphaned
)

func (ProcessingStatus) String added in v0.20.0

func (ps ProcessingStatus) String() string

type RequestTracker added in v0.17.6

type RequestTracker struct {
	// contains filtered or unexported fields
}

RequestTracker is an index of RequestTrackerItems indexed by execution result Index on result ID, incorporated block ID and chunk index. Is concurrency-safe.

func NewRequestTracker added in v0.17.6

func NewRequestTracker(headers storage.Headers, blackoutPeriodMin, blackoutPeriodMax int) *RequestTracker

NewRequestTracker instantiates a new RequestTracker with blackout periods between min and max seconds.

func (*RequestTracker) GetAllIds added in v0.17.6

func (rt *RequestTracker) GetAllIds() []flow.Identifier

GetAllIds returns all result IDs that we are indexing

func (*RequestTracker) PruneUpToHeight added in v0.19.0

func (rt *RequestTracker) PruneUpToHeight(height uint64) error

PruneUpToHeight remove all tracker items for blocks whose height is strictly smaller that height. Note: items for blocks at height are retained. After pruning, items for blocks below the given height are dropped.

Monotonicity Requirement: The pruned height cannot decrease, as we cannot recover already pruned elements. If `height` is smaller than the previous value, the previous value is kept and the sentinel mempool.BelowPrunedThresholdError is returned.

func (*RequestTracker) Remove added in v0.17.6

func (rt *RequestTracker) Remove(resultIDs ...flow.Identifier)

Remove removes all entries pertaining to an execution result

func (*RequestTracker) TryUpdate added in v0.19.0

func (rt *RequestTracker) TryUpdate(result *flow.ExecutionResult, incorporatedBlockID flow.Identifier, chunkIndex uint64) (RequestTrackerItem, bool, error)

TryUpdate tries to update tracker item if it's not in blackout period. Returns the tracker item for a specific chunk (creates it if it doesn't exists) and whenever request item was successfully updated or not. Since RequestTracker prunes items by height it can't accept items for height lower than cached lowest height. If height of executed block pointed by execution result is smaller than the lowest height, sentinel mempool.BelowPrunedThresholdError is returned. In case execution result points to unknown executed block exception will be returned.

type RequestTrackerItem added in v0.17.6

type RequestTrackerItem struct {
	Requests    uint
	NextTimeout time.Time
	// contains filtered or unexported fields
}

RequestTrackerItem is an object that keeps track of how many times a request has been made, as well as the time until a new request can be made. It is not concurrency-safe.

func NewRequestTrackerItem added in v0.17.6

func NewRequestTrackerItem(blackoutPeriodMin, blackoutPeriodMax int) (RequestTrackerItem, error)

NewRequestTrackerItem instantiates a new RequestTrackerItem where the NextTimeout is evaluated to the current time plus a random blackout period contained between min and max.

func (RequestTrackerItem) IsBlackout added in v0.17.6

func (i RequestTrackerItem) IsBlackout() bool

func (RequestTrackerItem) Update added in v0.17.6

Update creates a _new_ RequestTrackerItem with incremented request number and updated NextTimeout. No errors are expected during normal operation.

type SignatureCollector added in v0.17.6

type SignatureCollector struct {
	// contains filtered or unexported fields
}

SignatureCollector contains a set of of signatures from verifiers attesting to the validity of an execution result chunk. NOT concurrency safe. TODO: this will be replaced with stateful BLS aggregation

func NewSignatureCollector added in v0.17.6

func NewSignatureCollector() SignatureCollector

NewSignatureCollector instantiates a new SignatureCollector

func (*SignatureCollector) Add added in v0.17.6

func (c *SignatureCollector) Add(signerID flow.Identifier, signature crypto.Signature)

Add appends a signature. Only the _first_ signature is retained for each signerID.

func (*SignatureCollector) BySigner added in v0.17.6

func (c *SignatureCollector) BySigner(signerID flow.Identifier) (*crypto.Signature, bool)

BySigner returns a signer's signature if it exists

func (*SignatureCollector) HasSigned added in v0.17.6

func (c *SignatureCollector) HasSigned(signerID flow.Identifier) bool

HasSigned checks if signer has already provided a signature

func (*SignatureCollector) NumberSignatures added in v0.17.6

func (c *SignatureCollector) NumberSignatures() uint

NumberSignatures returns the number of stored (distinct) signatures

func (*SignatureCollector) ToAggregatedSignature added in v0.17.6

func (c *SignatureCollector) ToAggregatedSignature() flow.AggregatedSignature

ToAggregatedSignature generates an aggregated signature from all signatures in the SignatureCollector

type VerifyingAssignmentCollector added in v0.20.0

type VerifyingAssignmentCollector struct {
	AssignmentCollectorBase
	// contains filtered or unexported fields
}

VerifyingAssignmentCollector Context:

  • When the same result is incorporated in multiple different forks, unique verifier assignment is determined for each fork.
  • The assignment collector is intended to encapsulate the known assignments for a particular execution result.

VerifyingAssignmentCollector has a strict ordering of processing, before processing approvals at least one incorporated result has to be processed. VerifyingAssignmentCollector takes advantage of internal caching to speed up processing approvals for different assignments VerifyingAssignmentCollector is responsible for validating approvals on result-level (checking signature, identity).

func NewVerifyingAssignmentCollector added in v0.20.0

func NewVerifyingAssignmentCollector(collectorBase AssignmentCollectorBase) (*VerifyingAssignmentCollector, error)

NewVerifyingAssignmentCollector instantiates a new VerifyingAssignmentCollector. All errors are unexpected and potential symptoms of internal bugs or state corruption (fatal).

func (*VerifyingAssignmentCollector) CheckEmergencySealing added in v0.20.0

func (ac *VerifyingAssignmentCollector) CheckEmergencySealing(observer consensus.SealingObservation, finalizedBlockHeight uint64) error

CheckEmergencySealing checks the managed assignments whether their result can be emergency sealed. Seals the results where possible. It returns error when running into any exception It returns nil when it's done the checking regardless whether there is any results being emergency sealed or not

func (*VerifyingAssignmentCollector) ProcessApproval added in v0.20.0

func (ac *VerifyingAssignmentCollector) ProcessApproval(approval *flow.ResultApproval) error

ProcessApproval ingests Result Approvals and triggers sealing of execution result when sufficient approvals have arrived. Error Returns:

  • nil in case of success (outdated approvals might be silently discarded)
  • engine.InvalidInputError if the result approval is invalid
  • any other errors might be symptoms of bugs or internal state corruption (fatal)

func (*VerifyingAssignmentCollector) ProcessIncorporatedResult added in v0.20.0

func (ac *VerifyingAssignmentCollector) ProcessIncorporatedResult(incorporatedResult *flow.IncorporatedResult) error

ProcessIncorporatedResult starts tracking the approval for IncorporatedResult. Method is idempotent. Error Returns:

  • no errors expected during normal operation; errors might be symptoms of bugs or internal state corruption (fatal)

func (*VerifyingAssignmentCollector) ProcessingStatus added in v0.20.0

func (ac *VerifyingAssignmentCollector) ProcessingStatus() ProcessingStatus

func (*VerifyingAssignmentCollector) RequestMissingApprovals added in v0.20.0

func (ac *VerifyingAssignmentCollector) RequestMissingApprovals(observation consensus.SealingObservation, maxHeightForRequesting uint64) (uint, error)

RequestMissingApprovals traverses all collectors and requests missing approval for every chunk that didn't get enough approvals from verifiers. Returns number of requests made and error in case something goes wrong.

Directories

Path Synopsis

Jump to

Keyboard shortcuts

? : This menu
/ : Search site
f or F : Jump to
y or Y : Canonical URL