ouroboros-consensus-0.18.0.0: Consensus layer for the Ouroboros blockchain protocol
Safe HaskellSafe-Inferred
LanguageHaskell2010

Ouroboros.Consensus.Storage.ChainDB.API

Synopsis

Main ChainDB API

data ChainDB m blk Source #

The chain database

The chain database provides a unified interface on top of:

  • The ImmutableDB, storing the part of the chain that can't roll back.
  • The VolatileDB, storing the blocks near the tip of the chain, possibly in multiple competing forks.
  • The LedgerDB, storing snapshots of the ledger state for blocks in the ImmutableDB (and in-memory snapshots for the rest).

In addition to providing a unifying interface on top of these disparate components, the main responsibilities that the ChainDB itself has are:

  • Chain selection (on initialization and whenever a block is added)
  • Trigger full recovery whenever we detect disk failure in any component
  • Provide iterators across fixed fragments of the current chain
  • Provide followers that track the status of the current chain

The ChainDB instantiates all the various type parameters of these databases to conform to the unified interface we provide here.

Constructors

ChainDB 

Fields

  • addBlockAsyncInvalidBlockPunishment m → blk → m (AddBlockPromise m blk)

    Add a block to the heap of blocks

    We do not assume that the block is valid (under the legder rules); it is the responsibility of the Chain DB itself to only select chains that are valid.

    Conversely, the caller cannot assume that the new block will be added to the current chain; even if the block is valid, it will not become part of the chain if there are other chains available that are preferred by the consensus algorithm (typically, longer chains).

    This function typically returns immediately, yielding a AddBlockPromise which can be used to wait for the result. You can use addBlock to add the block synchronously.

    NOTE: back pressure can be applied when overloaded.

  • chainSelAsync ∷ m ()

    Trigger reprocessing of blocks postponed by the LoE.

  • getCurrentChainSTM m (AnchoredFragment (Header blk))

    Get the current chain fragment

    Suppose the current chain is

    a -> b -> c -> d -> e -> f

    and suppose k = 2; this means that the most distant fork we can switch to is something like

    a -> b -> c -> d -> e' -> f'

    The fragment we return will be [e, f], anchored at d. In other words, the length of the fragment will under normal circumstances be exactly k blocks long. It may be shorter if

    • We are near genesis The anchor will be the genesis point (which does not correspond to an actual block)
    • The volatile DB suffered some data loss Typically (but not necessarily) the volatile DB will not be empty and the anchor will be pointing to the tip of the immutable DB.

    POSTCONDITION: The Chain DB will be able to switch to any fork starting from the anchor of the returned fragment or any subsequent block (provided the new fork is at least of the same length as the old).

    NOTE: A direct consequence of this guarantee is that the anchor of the fragment will move as the chain grows.

  • getLedgerDBSTM m (LedgerDB' blk)

    Return the LedgerDB containing the last k ledger states.

  • getTipBlock ∷ m (Maybe blk)

    Get block at the tip of the chain, if one exists

    Returns Nothing if the database is empty.

  • getTipHeader ∷ m (Maybe (Header blk))

    Get header at the tip of the chain

    NOTE: Calling getTipHeader is cheaper than getTipBlock and then extracting the header: most of the time the header at the tip is actually in memory, whereas the block never is.

    Returns Nothing if the database is empty.

  • getTipPointSTM m (Point blk)

    Get point of the tip of the chain

    Will return genesisPoint if the database is empty; if the current chain fragment is empty due to data loss in the volatile DB, getTipPoint will return the tip of the immutable DB.

  • getBlockComponent ∷ ∀ b. BlockComponent blk b → RealPoint blk → m (Maybe b)

    Get the given component(s) of the block at the specified point. If there is no block at the given point, Nothing is returned.

  • getIsFetchedSTM m (Point blk → Bool)

    Return membership check function for recent blocks

    This check is only reliable for blocks up to k away from the tip. For blocks older than that the results should be regarded as non-deterministic.

  • getIsValidSTM m (RealPoint blk → Maybe Bool)

    Return a function that tells whether a block is known to be valid or invalid.

    The function will return:

    • Just True: for blocks in the volatile DB that have been validated and were found to be valid. All blocks in the current chain fragment (i.e., getCurrentChain) are valid.
    • Just False: for blocks in the volatile DB that have been validated and were found to be invalid.
    • Nothing: for blocks not or no longer in the volatile DB, whether they are valid or not, including blocks in the immutable DB. Also for blocks in the volatile DB that haven't been validated (yet), e.g., because they are disconnected from the current chain or they are part of a shorter fork.
  • getMaxSlotNoSTM m MaxSlotNo

    Get the highest slot number stored in the ChainDB.

    Note that the corresponding block doesn't have to be part of the current chain, it could be part of some fork, or even be a disconnected block.

  • stream ∷ ∀ b. ResourceRegistry m → BlockComponent blk b → StreamFrom blk → StreamTo blk → m (Either (UnknownRange blk) (Iterator m blk b))

    Stream blocks

    Streaming is not restricted to the current fork, but there must be an unbroken path from the starting point to the end point /at the time of initialization/ of the iterator. Once the iterator has been initialized, it will not be affected by subsequent calls to addBlock. To track the current chain, use a Follower instead.

    Streaming blocks older than k is permitted, but only when they are part of the current fork (at the time of initialization). Streaming a fork that forks off more than k blocks in the past is not permitted and an UnknownRange error will be returned in that case.

    The iterator does have a limited lifetime, however. The chain DB internally partitions the chain into an " immutable " part and a " volatile " part, moving blocks from the volatile DB to the immutable DB when they become more than k deep into the chain. When a block with slot number n is added to the immutble DB, a time delay t kicks in; after that time delay expires, all blocks older than n may be removed from the volatile DB, /including any blocks that happen to live on other forks/ (since those forks must now, by definition, be too distant). This time delay t also provides a worst-case bound for the lifetime of the iterator: if the iterator traverses a chain that forks off from our current chain at the tip of the immutable DB, then the first block on that fork will become unavailable as soon as another block is pushed to the current chain and the subsequent time delay expires.

    Note: although blocks are moved from the volatile DB to the immutable DB after they have become k deep into the chain, due to data corruption the suffix of the chain in the volatile DB might be shorter than k. The immutable DB always determines the maximum rollback, which may therefore be shorter than k under such circumstances. In addition, streaming blocks which aren't on the current fork is permitted, but the oldest volatile block must fit on to the tip of the immutable DB.

    When the given bounds are nonsensical, an InvalidIteratorRange is thrown.

    When the given bounds are not part of the chain DB, an UnknownRange error is returned.

    To stream all blocks from the current chain, use streamAll, as it correctly handles an empty ChainDB.

  • newFollower ∷ ∀ b. ResourceRegistry m → ChainTypeBlockComponent blk b → m (Follower m blk b)

    Chain follower

    A chain follower is an iterator that tracks the state of the current chain: calling next on the iterator will either give you the next block header, or (if we have switched to a fork) the instruction to rollback.

    The tracking iterator starts at genesis (see also trackForward).

    This is intended for use by chain consumers to reliably follow a chain, desipite the chain being volatile.

    Examples of users: * The server side of the chain sync mini-protocol for the node-to-node protocol using headers and the block size. * The server side of the chain sync mini-protocol for the node-to-client protocol using blocks.

  • getIsInvalidBlockSTM m (WithFingerprint (HeaderHash blk → Maybe (InvalidBlockReason blk)))

    Function to check whether a block is known to be invalid.

    Blocks unknown to the ChainDB will result in Nothing.

    If the hash corresponds to a block that is known to be invalid, but is now older than k, this function may return Nothing.

    Whenever a new invalid block is added, the Fingerprint will be changed. This is useful when "watching" this function in a transaction.

    Note that when invalid blocks are garbage collected and thus no longer detected by this function, the Fingerprint doesn't have to change, since the function will not detect new invalid blocks.

    It might seem natural to have this function also return whether the ChainDB knows that a block is valid, thereby subsuming the getIsValid function and simplifying the API. However, this adds the overhead of checking whether the block is valid for blocks that are not known to be invalid that does not give useful information to current clients (ChainSync), since they are only interested in whether a block is known to be invalid. The extra information of whether a block is valid is only used for testing.

    In particular, this affects the watcher in bracketChainSyncClient, which rechecks the blocks in all candidate chains whenever a new invalid block is detected. These blocks are likely to be valid.

  • closeDB ∷ m ()
     
  • isOpenSTM m Bool

    Return True when the database is open.

    False when the database is closed.

getCurrentLedger ∷ (Monad (STM m), IsLedger (LedgerState blk)) ⇒ ChainDB m blk → STM m (ExtLedgerState blk) Source #

Get current ledger

getCurrentTip ∷ (Monad (STM m), HasHeader (Header blk)) ⇒ ChainDB m blk → STM m (Tip blk) Source #

getHeaderStateHistoryMonad (STM m) ⇒ ChainDB m blk → STM m (HeaderStateHistory blk) Source #

Get a HeaderStateHistory populated with the HeaderStates of the last k blocks of the current chain.

getImmutableLedgerMonad (STM m) ⇒ ChainDB m blk → STM m (ExtLedgerState blk) Source #

Get the immutable ledger, i.e., typically k blocks back.

getPastLedger ∷ (Monad (STM m), LedgerSupportsProtocol blk) ⇒ ChainDB m blk → Point blk → STM m (Maybe (ExtLedgerState blk)) Source #

Get the ledger for the given point.

When the given point is not among the last k blocks of the current chain (i.e., older than k or not on the current chain), Nothing is returned.

Adding a block

data AddBlockPromise m blk Source #

Constructors

AddBlockPromise 

Fields

  • blockWrittenToDiskSTM m Bool

    Use this STM transaction to wait until the block has been written to disk.

    Returns True when the block was written to disk or False when it was ignored, e.g., because it was older than k.

    If the STM transaction has returned True then getIsFetched will return True for the added block.

    NOTE: Even when the result is False, getIsFetched might still return True, e.g., the block was older than k, but it has been downloaded and stored on disk before.

  • blockProcessedSTM m (AddBlockResult blk)

    Use this STM transaction to wait until the block has been processed: the block has been written to disk and chain selection has been performed for the block, unless the block is from the future.

    The ChainDB's tip after chain selection is returned. When this tip doesn't match the added block, it doesn't necessarily mean the block wasn't adopted. We might have adopted a longer chain of which the added block is a part, but not the tip.

    It returns FailedToAddBlock if the thread adding the block died.

    NOTE: When the block is from the future, chain selection for the block won't be performed until the block is no longer in the future, which might take some time. For that reason, this transaction will not wait for chain selection of a block from the future. It will return the current tip of the ChainDB after writing the block to disk.

data AddBlockResult blk Source #

This is a wrapper type for blockProcessed function above.

As it is mentioned the SuccessfullyAddedBlock constructor will containt the ChainDB's tip after chain selection is returned.

The FailedToAddBlock case will be returned if the thread adding the block died.

Instances

Instances details
StandardHash blk ⇒ Show (AddBlockResult blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

showsPrecIntAddBlockResult blk → ShowS #

showAddBlockResult blk → String #

showList ∷ [AddBlockResult blk] → ShowS #

StandardHash blk ⇒ Eq (AddBlockResult blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

(==)AddBlockResult blk → AddBlockResult blk → Bool #

(/=)AddBlockResult blk → AddBlockResult blk → Bool #

addBlockIOLike m ⇒ ChainDB m blk → InvalidBlockPunishment m → blk → m (AddBlockResult blk) Source #

Add a block synchronously: wait until the block has been processed (see blockProcessed). The new tip of the ChainDB is returned unless the thread adding the block died, in that case FailedToAddBlock will be returned.

Note: this is a partial function, only to support tests.

addBlockWaitWrittenToDiskIOLike m ⇒ ChainDB m blk → InvalidBlockPunishment m → blk → m Bool Source #

Add a block synchronously: wait until the block has been written to disk (see blockWrittenToDisk).

addBlock_IOLike m ⇒ ChainDB m blk → InvalidBlockPunishment m → blk → m () Source #

Add a block synchronously. Variant of addBlock that doesn't return the new tip of the ChainDB.

Note: this is a partial function, only to support tests.

triggerChainSelectionAsyncChainDB m blk → m () Source #

Alias for naming consistency. The short name was chosen to avoid a larger diff from alignment changes.

Serialised block/header with its point

data WithPoint blk b Source #

A b together with its Point.

The Point is needed because we often need to know the hash, slot, or point itself of the block or header in question, and we don't want to deserialise the block to obtain it.

Constructors

WithPoint 

Fields

Instances

Instances details
StandardHash blk ⇒ StandardHash (WithPoint blk b ∷ Type) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type HeaderHash (WithPoint blk b ∷ Type) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type HeaderHash (WithPoint blk b ∷ Type) = HeaderHash blk

BlockComponent

data BlockComponent blk a where Source #

Which component of the block to read from a database: the whole block, its header, its hash, the block size, ..., or combinations thereof.

NOTE: when requesting multiple components, we will not optimise/cache them.

Constructors

GetVerifiedBlockBlockComponent blk blk

Verify the integrity of the block by checking its signature and/or hashes. The interpreter should throw an exception when the block does not pass the check.

GetBlockBlockComponent blk blk 
GetRawBlockBlockComponent blk ByteString 
GetHeaderBlockComponent blk (Header blk) 
GetRawHeaderBlockComponent blk ByteString 
GetHashBlockComponent blk (HeaderHash blk) 
GetSlotBlockComponent blk SlotNo 
GetIsEBBBlockComponent blk IsEBB 
GetBlockSizeBlockComponent blk SizeInBytes 
GetHeaderSizeBlockComponent blk Word16 
GetNestedCtxtBlockComponent blk (SomeSecond (NestedCtxt Header) blk) 
GetPure ∷ a → BlockComponent blk a 
GetApplyBlockComponent blk (a → b) → BlockComponent blk a → BlockComponent blk b 

Instances

Instances details
Applicative (BlockComponent blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Methods

pure ∷ a → BlockComponent blk a #

(<*>)BlockComponent blk (a → b) → BlockComponent blk a → BlockComponent blk b #

liftA2 ∷ (a → b → c) → BlockComponent blk a → BlockComponent blk b → BlockComponent blk c #

(*>)BlockComponent blk a → BlockComponent blk b → BlockComponent blk b #

(<*)BlockComponent blk a → BlockComponent blk b → BlockComponent blk a #

Functor (BlockComponent blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Methods

fmap ∷ (a → b) → BlockComponent blk a → BlockComponent blk b #

(<$) ∷ a → BlockComponent blk b → BlockComponent blk a #

Support for tests

fromChain ∷ ∀ m blk. IOLike m ⇒ m (ChainDB m blk) → Chain blk → m (ChainDB m blk) Source #

toChain ∷ ∀ m blk. (HasCallStack, IOLike m, HasHeader blk) ⇒ ChainDB m blk → m (Chain blk) Source #

Iterator API

data Iterator m blk b Source #

Constructors

Iterator 

Fields

Instances

Instances details
Foldable m ⇒ Foldable (Iterator m blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

foldMonoid m0 ⇒ Iterator m blk m0 → m0 #

foldMapMonoid m0 ⇒ (a → m0) → Iterator m blk a → m0 #

foldMap'Monoid m0 ⇒ (a → m0) → Iterator m blk a → m0 #

foldr ∷ (a → b → b) → b → Iterator m blk a → b #

foldr' ∷ (a → b → b) → b → Iterator m blk a → b #

foldl ∷ (b → a → b) → b → Iterator m blk a → b #

foldl' ∷ (b → a → b) → b → Iterator m blk a → b #

foldr1 ∷ (a → a → a) → Iterator m blk a → a #

foldl1 ∷ (a → a → a) → Iterator m blk a → a #

toListIterator m blk a → [a] #

nullIterator m blk a → Bool #

lengthIterator m blk a → Int #

elemEq a ⇒ a → Iterator m blk a → Bool #

maximumOrd a ⇒ Iterator m blk a → a #

minimumOrd a ⇒ Iterator m blk a → a #

sumNum a ⇒ Iterator m blk a → a #

productNum a ⇒ Iterator m blk a → a #

Traversable m ⇒ Traversable (Iterator m blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

traverseApplicative f ⇒ (a → f b) → Iterator m blk a → f (Iterator m blk b) #

sequenceAApplicative f ⇒ Iterator m blk (f a) → f (Iterator m blk a) #

mapMMonad m0 ⇒ (a → m0 b) → Iterator m blk a → m0 (Iterator m blk b) #

sequenceMonad m0 ⇒ Iterator m blk (m0 a) → m0 (Iterator m blk a) #

Functor m ⇒ Functor (Iterator m blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

fmap ∷ (a → b) → Iterator m blk a → Iterator m blk b #

(<$) ∷ a → Iterator m blk b → Iterator m blk a #

data IteratorResult blk b Source #

Constructors

IteratorExhausted 
IteratorResult b 
IteratorBlockGCed (RealPoint blk)

The block that was supposed to be streamed was garbage-collected from the VolatileDB, but not added to the ImmutableDB.

This will only happen when streaming very old forks very slowly.

The following example illustrates a situation in which an iterator result could be a IteratorBlockGCed value. Suppose we start with an iterator positioned at block c, where [[ x ]] denotes a block in the immutable DB:

                          iterator i
                            ↓
... ⟶ [[ a ]] → [[ b ]] → [ c ] -> [ d ]
──────────────────────╯   ╰────────────╯
  Immutable DB             Current chain

Suppose we switch to a longer fork that branches off from the immutable tip ('[[b]]').

                           iterator i
                            ↓
... ⟶ [[ a ]] → [[ b ]] → [ c ] -> [ d ]
──────────────────────╯│
   Immutable DB        │
                       ╰-→ [ e ] -> [ f ] -> [ g ]
                           ╰─────────────────────╯
                                 Current chain

Assume k=2. This means that block e is the new immutable tip. If we would call iteratorNext on i after block e is copied to the immutable DB and c and d are garbage collected, then we will get IteratorBlockGCed.

Instances

Instances details
Foldable (IteratorResult blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

foldMonoid m ⇒ IteratorResult blk m → m #

foldMapMonoid m ⇒ (a → m) → IteratorResult blk a → m #

foldMap'Monoid m ⇒ (a → m) → IteratorResult blk a → m #

foldr ∷ (a → b → b) → b → IteratorResult blk a → b #

foldr' ∷ (a → b → b) → b → IteratorResult blk a → b #

foldl ∷ (b → a → b) → b → IteratorResult blk a → b #

foldl' ∷ (b → a → b) → b → IteratorResult blk a → b #

foldr1 ∷ (a → a → a) → IteratorResult blk a → a #

foldl1 ∷ (a → a → a) → IteratorResult blk a → a #

toListIteratorResult blk a → [a] #

nullIteratorResult blk a → Bool #

lengthIteratorResult blk a → Int #

elemEq a ⇒ a → IteratorResult blk a → Bool #

maximumOrd a ⇒ IteratorResult blk a → a #

minimumOrd a ⇒ IteratorResult blk a → a #

sumNum a ⇒ IteratorResult blk a → a #

productNum a ⇒ IteratorResult blk a → a #

Traversable (IteratorResult blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

traverseApplicative f ⇒ (a → f b) → IteratorResult blk a → f (IteratorResult blk b) #

sequenceAApplicative f ⇒ IteratorResult blk (f a) → f (IteratorResult blk a) #

mapMMonad m ⇒ (a → m b) → IteratorResult blk a → m (IteratorResult blk b) #

sequenceMonad m ⇒ IteratorResult blk (m a) → m (IteratorResult blk a) #

Functor (IteratorResult blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

fmap ∷ (a → b) → IteratorResult blk a → IteratorResult blk b #

(<$) ∷ a → IteratorResult blk b → IteratorResult blk a #

(Show blk, Show b, StandardHash blk) ⇒ Show (IteratorResult blk b) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

showsPrecIntIteratorResult blk b → ShowS #

showIteratorResult blk b → String #

showList ∷ [IteratorResult blk b] → ShowS #

(Eq blk, Eq b, StandardHash blk) ⇒ Eq (IteratorResult blk b) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

(==)IteratorResult blk b → IteratorResult blk b → Bool #

(/=)IteratorResult blk b → IteratorResult blk b → Bool #

data StreamFrom blk Source #

The lower bound for an iterator

Hint: use StreamFromExclusive genesisPoint to start streaming from Genesis.

Instances

Instances details
Generic (StreamFrom blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Associated Types

type Rep (StreamFrom blk) ∷ TypeType #

Methods

fromStreamFrom blk → Rep (StreamFrom blk) x #

toRep (StreamFrom blk) x → StreamFrom blk #

StandardHash blk ⇒ Show (StreamFrom blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Methods

showsPrecIntStreamFrom blk → ShowS #

showStreamFrom blk → String #

showList ∷ [StreamFrom blk] → ShowS #

StandardHash blk ⇒ Eq (StreamFrom blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Methods

(==)StreamFrom blk → StreamFrom blk → Bool #

(/=)StreamFrom blk → StreamFrom blk → Bool #

(StandardHash blk, Typeable blk) ⇒ NoThunks (StreamFrom blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

type Rep (StreamFrom blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

type Rep (StreamFrom blk) = D1 ('MetaData "StreamFrom" "Ouroboros.Consensus.Storage.Common" "ouroboros-consensus-0.18.0.0-inplace" 'False) (C1 ('MetaCons "StreamFromInclusive" 'PrefixI 'False) (S1 ('MetaSel ('NothingMaybe Symbol) 'NoSourceUnpackedness 'SourceStrict 'DecidedStrict) (Rec0 (RealPoint blk))) :+: C1 ('MetaCons "StreamFromExclusive" 'PrefixI 'False) (S1 ('MetaSel ('NothingMaybe Symbol) 'NoSourceUnpackedness 'SourceStrict 'DecidedStrict) (Rec0 (Point blk))))

newtype StreamTo blk Source #

Constructors

StreamToInclusive (RealPoint blk) 

Instances

Instances details
Generic (StreamTo blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Associated Types

type Rep (StreamTo blk) ∷ TypeType #

Methods

fromStreamTo blk → Rep (StreamTo blk) x #

toRep (StreamTo blk) x → StreamTo blk #

StandardHash blk ⇒ Show (StreamTo blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Methods

showsPrecIntStreamTo blk → ShowS #

showStreamTo blk → String #

showList ∷ [StreamTo blk] → ShowS #

StandardHash blk ⇒ Eq (StreamTo blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

Methods

(==)StreamTo blk → StreamTo blk → Bool #

(/=)StreamTo blk → StreamTo blk → Bool #

(StandardHash blk, Typeable blk) ⇒ NoThunks (StreamTo blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

type Rep (StreamTo blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.Common

type Rep (StreamTo blk) = D1 ('MetaData "StreamTo" "Ouroboros.Consensus.Storage.Common" "ouroboros-consensus-0.18.0.0-inplace" 'True) (C1 ('MetaCons "StreamToInclusive" 'PrefixI 'False) (S1 ('MetaSel ('NothingMaybe Symbol) 'NoSourceUnpackedness 'NoSourceStrictness 'DecidedLazy) (Rec0 (RealPoint blk))))

data UnknownRange blk Source #

Constructors

MissingBlock (RealPoint blk)

The block at the given point was not found in the ChainDB.

ForkTooOld (StreamFrom blk)

The requested range forks off too far in the past, i.e. it doesn't fit on the tip of the ImmutableDB.

Instances

Instances details
StandardHash blk ⇒ Show (UnknownRange blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

showsPrecIntUnknownRange blk → ShowS #

showUnknownRange blk → String #

showList ∷ [UnknownRange blk] → ShowS #

StandardHash blk ⇒ Eq (UnknownRange blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

(==)UnknownRange blk → UnknownRange blk → Bool #

(/=)UnknownRange blk → UnknownRange blk → Bool #

emptyIteratorMonad m ⇒ Iterator m blk b Source #

An iterator that is immediately exhausted.

streamAll ∷ (MonadSTM m, HasHeader blk, HasCallStack) ⇒ ChainDB m blk → ResourceRegistry m → BlockComponent blk b → m (Iterator m blk b) Source #

Stream all blocks from the current chain.

streamFrom ∷ (MonadSTM m, HasHeader blk, HasCallStack) ⇒ StreamFrom blk → ChainDB m blk → ResourceRegistry m → BlockComponent blk b → m (Iterator m blk b) Source #

Stream blocks from the given point up to the tip from the current chain.

To stream all blocks from the current chain from the ChainDB, one would use StreamFromExclusive genesisPoint as the lower bound and StreamToInclusive tip as the upper bound where tip is retrieved with getTipPoint.

However, when the ChainDB is empty, tip will be genesisPoint too, in which case the bounds don't make sense. This function correctly handles this case.

Note that this is not a Follower, so the stream will not include blocks that are added to the current chain after starting the stream.

traverseIteratorMonad m ⇒ (b → m b') → Iterator m blk b → Iterator m blk b' Source #

Variant of traverse instantiated to Iterator m blk that executes the monadic function when calling iteratorNext.

validBoundsStandardHash blk ⇒ StreamFrom blk → StreamTo blk → Bool Source #

Check whether the bounds make sense

An example of bounds that don't make sense:

StreamFromExclusive (BlockPoint 3 ..)
StreamToInclusive   (RealPoint  3 ..)

This function does not check whether the bounds correspond to existing blocks.

Invalid block reason

data InvalidBlockReason blk Source #

The reason why a block is invalid.

Constructors

ValidationError !(ExtValidationError blk)

The ledger found the block to be invalid.

InFutureExceedsClockSkew !(RealPoint blk)

The block's slot is in the future, exceeding the allowed clock skew.

Possible causes, order by decreasing likelihood:

  1. Our clock is behind (significantly more likely than the others)
  2. Their clock is ahead
  3. It's intentional, i.e., an attack

Instances

Instances details
Generic (InvalidBlockReason blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Associated Types

type Rep (InvalidBlockReason blk) ∷ TypeType #

LedgerSupportsProtocol blk ⇒ Show (InvalidBlockReason blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

LedgerSupportsProtocol blk ⇒ Eq (InvalidBlockReason blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

LedgerSupportsProtocol blk ⇒ NoThunks (InvalidBlockReason blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type Rep (InvalidBlockReason blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type Rep (InvalidBlockReason blk) = D1 ('MetaData "InvalidBlockReason" "Ouroboros.Consensus.Storage.ChainDB.API" "ouroboros-consensus-0.18.0.0-inplace" 'False) (C1 ('MetaCons "ValidationError" 'PrefixI 'False) (S1 ('MetaSel ('NothingMaybe Symbol) 'NoSourceUnpackedness 'SourceStrict 'DecidedStrict) (Rec0 (ExtValidationError blk))) :+: C1 ('MetaCons "InFutureExceedsClockSkew" 'PrefixI 'False) (S1 ('MetaSel ('NothingMaybe Symbol) 'NoSourceUnpackedness 'SourceStrict 'DecidedStrict) (Rec0 (RealPoint blk))))

Followers

data ChainType Source #

Chain type

Followers can choose to track changes to the "normal" SelectedChain, or track the TentativeChain, which might contain a pipelineable header at the tip.

Instances

Instances details
Generic ChainType Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Associated Types

type Rep ChainTypeTypeType #

Methods

fromChainTypeRep ChainType x #

toRep ChainType x → ChainType #

Show ChainType Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

showsPrecIntChainTypeShowS #

showChainTypeString #

showList ∷ [ChainType] → ShowS #

Eq ChainType Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

(==)ChainTypeChainTypeBool #

(/=)ChainTypeChainTypeBool #

type Rep ChainType Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type Rep ChainType = D1 ('MetaData "ChainType" "Ouroboros.Consensus.Storage.ChainDB.API" "ouroboros-consensus-0.18.0.0-inplace" 'False) (C1 ('MetaCons "SelectedChain" 'PrefixI 'False) (U1TypeType) :+: C1 ('MetaCons "TentativeChain" 'PrefixI 'False) (U1TypeType))

data Follower m blk a Source #

Follower

Unlike an Iterator, which is used to request a static segment of the current chain or a recent fork, a follower is used to follow the current chain either from the start or from a given point.

Unlike an Iterator, a Follower is dynamic, that is, it will follow the chain when it grows or forks.

A follower is pull-based, which avoids the neeed to have a growing queue of changes to the chain on the server side in case the client is slower.

A follower always has an implicit position associated with it. The followerInstruction and followerInstructionBlocking operations request the next ChainUpdate wrt the follower's implicit position.

The type parameter a will be instantiated with blk or Header blk.

Constructors

Follower 

Fields

  • followerInstruction ∷ m (Maybe (ChainUpdate blk a))

    The next chain update (if one exists)

    The AddBlock instruction (see ChainUpdate) indicates that, to follow the current chain, the follower should extend its chain with the given block component (which will be a value of type a).

    The RollBack instruction indicates that the follower should perform a rollback by first backtracking to a certain point.

    If a follower should switch to a fork, then it will first receive a RollBack instruction followed by as many AddBlock as necessary to reach the tip of the new chain.

    When the follower's (implicit) position is in the immutable part of the chain, no rollback instructions will be encountered.

    Not in STM because might have to read the blocks or headers from disk.

    We may roll back more than k, but only in case of data loss.

  • followerInstructionBlocking ∷ m (ChainUpdate blk a)

    Blocking version of followerInstruction

  • followerForward ∷ [Point blk] → m (Maybe (Point blk))

    Move the follower forward

    Must be given a list of points in order of preference; the iterator will move forward to the first point on the list that is on the current chain. Returns Nothing if the iterator did not move, or the new point otherwise.

    When successful, the first call to followerInstruction after followerForward will be a RollBack to the point returned by followerForward.

    Cannot live in STM because the points specified might live in the immutable DB.

  • followerClose ∷ m ()

    Close the follower.

    Idempotent.

    After closing, all other operations on the follower will throw ClosedFollowerError.

Instances

Instances details
Functor m ⇒ Functor (Follower m blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

fmap ∷ (a → b) → Follower m blk a → Follower m blk b #

(<$) ∷ a → Follower m blk b → Follower m blk a #

traverseFollowerMonad m ⇒ (b → m b') → Follower m blk b → Follower m blk b' Source #

Variant of traverse instantiated to Follower m blk that executes the monadic function when calling followerInstruction and followerInstructionBlocking.

Recovery

data ChainDbFailure blk Source #

Database failure

This exception wraps any kind of unexpected problem with the on-disk storage of the chain.

The various constructors only serve to give more detailed information about what went wrong, in case sysadmins want to investigate the disk failure. The Chain DB itself does not differentiate; all disk failures are treated equal and all trigger the same recovery procedure.

Constructors

LgrDbFailure FsError

The ledger DB threw a file-system error

ChainDbMissingBlock (RealPoint blk)

Block missing from the chain DB

Thrown when we are not sure in which DB the block should have been.

data IsEBB Source #

Whether a block is an Epoch Boundary Block (EBB)

See Ouroboros.Storage.ImmutableDB.API for a discussion of EBBs. Key idiosyncracies:

  • An EBB carries no unique information.
  • An EBB has the same BlockNo as its predecessor.
  • EBBs are vestigial. As of Shelley, nodes no longer forge EBBs: they are only a legacy/backwards-compatibility concern.

Constructors

IsEBB 
IsNotEBB 

Instances

Instances details
Generic IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

Associated Types

type Rep IsEBBTypeType #

Methods

fromIsEBBRep IsEBB x #

toRep IsEBB x → IsEBB #

Show IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

Methods

showsPrecIntIsEBBShowS #

showIsEBBString #

showList ∷ [IsEBB] → ShowS #

Eq IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

Methods

(==)IsEBBIsEBBBool #

(/=)IsEBBIsEBBBool #

NoThunks IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

Condense IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

Methods

condenseIsEBBString Source #

Serialise IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

type Rep IsEBB Source # 
Instance details

Defined in Ouroboros.Consensus.Block.EBB

type Rep IsEBB = D1 ('MetaData "IsEBB" "Ouroboros.Consensus.Block.EBB" "ouroboros-consensus-0.18.0.0-inplace" 'False) (C1 ('MetaCons "IsEBB" 'PrefixI 'False) (U1TypeType) :+: C1 ('MetaCons "IsNotEBB" 'PrefixI 'False) (U1TypeType))

Exceptions

data ChainDbError blk Source #

Database error

Thrown upon incorrect use: invalid input.

Constructors

ClosedDBError PrettyCallStack

The ChainDB is closed.

This will be thrown when performing any operation on the ChainDB except for isOpen and closeDB. The CallStack of the operation on the ChainDB is included in the error.

ClosedFollowerError

The follower is closed.

This will be thrown when performing any operation on a closed followers, except for followerClose.

InvalidIteratorRange (StreamFrom blk) (StreamTo blk)

When there is no chain/fork that satisfies the bounds passed to streamBlocks.

  • The lower and upper bound are not on the same chain.
  • The bounds don't make sense, e.g., the lower bound starts after the upper bound, or the lower bound starts from genesis, inclusive.

Instances

Instances details
(Typeable blk, StandardHash blk) ⇒ Exception (ChainDbError blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

(Typeable blk, StandardHash blk) ⇒ Show (ChainDbError blk) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

showsPrecIntChainDbError blk → ShowS #

showChainDbError blk → String #

showList ∷ [ChainDbError blk] → ShowS #

Genesis

data LoE a Source #

The Limit on Eagerness is a mechanism for keeping ChainSel from advancing the current selection in the case of competing chains.

The Limit on Eagerness prevents the selection of the node from extending more than k blocks after the youngest block that is present on all candidate fragments.

It requires a resolution mechanism to prevent indefinite stalling, which is implemented by the Genesis Density Disconnection governor, a component that implements an UpdateLoEFrag that disconnects from peers with forks it considers inferior.

This type indicates whether the feature is enabled, and contains a value if it is.

Constructors

LoEDisabled

The LoE is disabled, so ChainSel will not keep the selection from advancing.

LoEEnabled a

The LoE is enabled, using the security parameter k as the limit. When the selection's tip is k blocks after the earliest intersection of of all candidate fragments, ChainSel will not add new blocks to the selection.

Instances

Instances details
Foldable LoE Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

foldMonoid m ⇒ LoE m → m #

foldMapMonoid m ⇒ (a → m) → LoE a → m #

foldMap'Monoid m ⇒ (a → m) → LoE a → m #

foldr ∷ (a → b → b) → b → LoE a → b #

foldr' ∷ (a → b → b) → b → LoE a → b #

foldl ∷ (b → a → b) → b → LoE a → b #

foldl' ∷ (b → a → b) → b → LoE a → b #

foldr1 ∷ (a → a → a) → LoE a → a #

foldl1 ∷ (a → a → a) → LoE a → a #

toListLoE a → [a] #

nullLoE a → Bool #

lengthLoE a → Int #

elemEq a ⇒ a → LoE a → Bool #

maximumOrd a ⇒ LoE a → a #

minimumOrd a ⇒ LoE a → a #

sumNum a ⇒ LoE a → a #

productNum a ⇒ LoE a → a #

Traversable LoE Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

traverseApplicative f ⇒ (a → f b) → LoE a → f (LoE b) #

sequenceAApplicative f ⇒ LoE (f a) → f (LoE a) #

mapMMonad m ⇒ (a → m b) → LoE a → m (LoE b) #

sequenceMonad m ⇒ LoE (m a) → m (LoE a) #

Functor LoE Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

fmap ∷ (a → b) → LoE a → LoE b #

(<$) ∷ a → LoE b → LoE a #

Generic (LoE a) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Associated Types

type Rep (LoE a) ∷ TypeType #

Methods

fromLoE a → Rep (LoE a) x #

toRep (LoE a) x → LoE a #

Show a ⇒ Show (LoE a) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

showsPrecIntLoE a → ShowS #

showLoE a → String #

showList ∷ [LoE a] → ShowS #

Eq a ⇒ Eq (LoE a) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

Methods

(==)LoE a → LoE a → Bool #

(/=)LoE a → LoE a → Bool #

NoThunks a ⇒ NoThunks (LoE a) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type Rep (LoE a) Source # 
Instance details

Defined in Ouroboros.Consensus.Storage.ChainDB.API

type Rep (LoE a) = D1 ('MetaData "LoE" "Ouroboros.Consensus.Storage.ChainDB.API" "ouroboros-consensus-0.18.0.0-inplace" 'False) (C1 ('MetaCons "LoEDisabled" 'PrefixI 'False) (U1TypeType) :+: C1 ('MetaCons "LoEEnabled" 'PrefixI 'False) (S1 ('MetaSel ('NothingMaybe Symbol) 'NoSourceUnpackedness 'NoSourceStrictness 'DecidedLazy) (Rec0 a)))