Add selected chain store and optimize block locator with it (#1394)

* Add selected chain store and optimize block locator with it

* Fix build error

* Fix comments

* Fix IsStaged

* Rename CalculateSelectedParentChainChanges to CalculateChainPath and SelectedParentChainChanges->SelectedChainPath

* Use binary.LittleEndian directly to allow compiler optimizations

* Remove boolean from HeadersSelectedChainStore interface

* Prevent endless loop in block locator
This commit is contained in:
Ori Newman 2021-01-11 15:51:45 +02:00 committed by GitHub
parent c7deda41c6
commit b8ca33d91d
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
31 changed files with 713 additions and 194 deletions

View File

@ -131,7 +131,7 @@ func (flow *handleRelayInvsFlow) findHighestSharedBlockHash(targetHash *external
for !lowHash.Equal(highHash) { for !lowHash.Equal(highHash) {
log.Debugf("Sending a blockLocator to %s between %s and %s", flow.peer, lowHash, highHash) log.Debugf("Sending a blockLocator to %s between %s and %s", flow.peer, lowHash, highHash)
blockLocator, err := flow.Domain().Consensus().CreateBlockLocator(lowHash, highHash, 0) blockLocator, err := flow.Domain().Consensus().CreateHeadersSelectedChainBlockLocator(lowHash, highHash)
if err != nil { if err != nil {
return nil, err return nil, err
} }
@ -169,6 +169,13 @@ func (flow *handleRelayInvsFlow) findHighestSharedBlockHash(targetHash *external
log.Debugf("The index of the highest hash in the original "+ log.Debugf("The index of the highest hash in the original "+
"blockLocator sent to %s is %d", flow.peer, highestHashIndex) "blockLocator sent to %s is %d", flow.peer, highestHashIndex)
// If the block locator contains only two adjacent chain blocks, the
// syncer will always find the same highest chain block, so to avoid
// an endless loop, we explicitly stop the loop in such situation.
if len(blockLocator) == 2 && highestHashIndex == 1 {
return highestHash, nil
}
locatorHashAboveHighestHash := highestHash locatorHashAboveHighestHash := highestHash
if highestHashIndex > 0 { if highestHashIndex > 0 {
locatorHashAboveHighestHash = blockLocator[highestHashIndex-1] locatorHashAboveHighestHash = blockLocator[highestHashIndex-1]

View File

@ -9,7 +9,7 @@ import (
// ConvertVirtualSelectedParentChainChangesToChainChangedNotificationMessage converts // ConvertVirtualSelectedParentChainChangesToChainChangedNotificationMessage converts
// VirtualSelectedParentChainChanges to VirtualSelectedParentChainChangedNotificationMessage // VirtualSelectedParentChainChanges to VirtualSelectedParentChainChangedNotificationMessage
func (ctx *Context) ConvertVirtualSelectedParentChainChangesToChainChangedNotificationMessage( func (ctx *Context) ConvertVirtualSelectedParentChainChangesToChainChangedNotificationMessage(
selectedParentChainChanges *externalapi.SelectedParentChainChanges) (*appmessage.VirtualSelectedParentChainChangedNotificationMessage, error) { selectedParentChainChanges *externalapi.SelectedChainPath) (*appmessage.VirtualSelectedParentChainChangedNotificationMessage, error) {
removedChainBlockHashes := make([]string, len(selectedParentChainChanges.Removed)) removedChainBlockHashes := make([]string, len(selectedParentChainChanges.Removed))
for i, removed := range selectedParentChainChanges.Removed { for i, removed := range selectedParentChainChanges.Removed {

View File

@ -45,6 +45,7 @@ type consensus struct {
reachabilityDataStore model.ReachabilityDataStore reachabilityDataStore model.ReachabilityDataStore
utxoDiffStore model.UTXODiffStore utxoDiffStore model.UTXODiffStore
finalityStore model.FinalityStore finalityStore model.FinalityStore
headersSelectedChainStore model.HeadersSelectedChainStore
} }
// BuildBlock builds a block over the current state, with the transactions // BuildBlock builds a block over the current state, with the transactions
@ -297,6 +298,14 @@ func (s *consensus) CreateBlockLocator(lowHash, highHash *externalapi.DomainHash
return s.syncManager.CreateBlockLocator(lowHash, highHash, limit) return s.syncManager.CreateBlockLocator(lowHash, highHash, limit)
} }
func (s *consensus) CreateHeadersSelectedChainBlockLocator(lowHash,
highHash *externalapi.DomainHash) (externalapi.BlockLocator, error) {
s.lock.Lock()
defer s.lock.Unlock()
return s.syncManager.CreateHeadersSelectedChainBlockLocator(lowHash, highHash)
}
func (s *consensus) FindNextBlockLocatorBoundaries(blockLocator externalapi.BlockLocator) (lowHash, highHash *externalapi.DomainHash, err error) { func (s *consensus) FindNextBlockLocatorBoundaries(blockLocator externalapi.BlockLocator) (lowHash, highHash *externalapi.DomainHash, err error) {
s.lock.Lock() s.lock.Lock()
defer s.lock.Unlock() defer s.lock.Unlock()
@ -327,7 +336,7 @@ func (s *consensus) IsValidPruningPoint(blockHash *externalapi.DomainHash) (bool
return s.pruningManager.IsValidPruningPoint(blockHash) return s.pruningManager.IsValidPruningPoint(blockHash)
} }
func (s *consensus) GetVirtualSelectedParentChainFromBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) { func (s *consensus) GetVirtualSelectedParentChainFromBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error) {
s.lock.Lock() s.lock.Lock()
defer s.lock.Unlock() defer s.lock.Unlock()

View File

@ -0,0 +1,13 @@
package binaryserialization
import "github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
// SerializeHash serializes hash to a slice of bytes
func SerializeHash(hash *externalapi.DomainHash) []byte {
return hash.ByteSlice()
}
// DeserializeHash a slice of bytes to a hash
func DeserializeHash(hashBytes []byte) (*externalapi.DomainHash, error) {
return externalapi.NewDomainHashFromByteSlice(hashBytes)
}

View File

@ -0,0 +1,15 @@
package binaryserialization
import "encoding/binary"
// SerializeChainBlockIndex serializes chain block index
func SerializeChainBlockIndex(index uint64) []byte {
var keyBytes [8]byte
binary.LittleEndian.PutUint64(keyBytes[:], index)
return keyBytes[:]
}
// DeserializeChainBlockIndex deserializes chain block index to uint64
func DeserializeChainBlockIndex(indexBytes []byte) uint64 {
return binary.LittleEndian.Uint64(indexBytes)
}

View File

@ -0,0 +1,230 @@
package headersselectedchainstore
import (
"encoding/binary"
"github.com/kaspanet/kaspad/domain/consensus/database/binaryserialization"
"github.com/kaspanet/kaspad/domain/consensus/model"
"github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
"github.com/kaspanet/kaspad/domain/consensus/utils/dbkeys"
"github.com/kaspanet/kaspad/domain/consensus/utils/lrucache"
"github.com/kaspanet/kaspad/domain/consensus/utils/lrucacheuint64tohash"
"github.com/kaspanet/kaspad/infrastructure/db/database"
"github.com/pkg/errors"
)
var bucketChainBlockHashByIndex = dbkeys.MakeBucket([]byte("chain-block-hash-by-index"))
var bucketChainBlockIndexByHash = dbkeys.MakeBucket([]byte("chain-block-index-by-hash"))
var highestChainBlockIndexKey = dbkeys.MakeBucket().Key([]byte("highest-chain-block-index"))
type headersSelectedChainStore struct {
stagingAddedByHash map[externalapi.DomainHash]uint64
stagingRemovedByHash map[externalapi.DomainHash]struct{}
stagingAddedByIndex map[uint64]*externalapi.DomainHash
stagingRemovedByIndex map[uint64]struct{}
cacheByIndex *lrucacheuint64tohash.LRUCache
cacheByHash *lrucache.LRUCache
cacheHighestChainBlockIndex uint64
}
// New instantiates a new HeadersSelectedChainStore
func New(cacheSize int) model.HeadersSelectedChainStore {
return &headersSelectedChainStore{
stagingAddedByHash: make(map[externalapi.DomainHash]uint64),
stagingRemovedByHash: make(map[externalapi.DomainHash]struct{}),
stagingAddedByIndex: make(map[uint64]*externalapi.DomainHash),
stagingRemovedByIndex: make(map[uint64]struct{}),
cacheByIndex: lrucacheuint64tohash.New(cacheSize),
cacheByHash: lrucache.New(cacheSize),
}
}
// Stage stages the given chain changes
func (hscs *headersSelectedChainStore) Stage(dbContext model.DBReader,
chainChanges *externalapi.SelectedChainPath) error {
if hscs.IsStaged() {
return errors.Errorf("can't stage when there's already staged data")
}
for _, blockHash := range chainChanges.Removed {
index, err := hscs.GetIndexByHash(dbContext, blockHash)
if err != nil {
return err
}
hscs.stagingRemovedByIndex[index] = struct{}{}
hscs.stagingRemovedByHash[*blockHash] = struct{}{}
}
currentIndex := uint64(0)
highestChainBlockIndex, exists, err := hscs.highestChainBlockIndex(dbContext)
if err != nil {
return err
}
if exists {
currentIndex = highestChainBlockIndex - uint64(len(chainChanges.Removed)) + 1
}
for _, blockHash := range chainChanges.Added {
hscs.stagingAddedByIndex[currentIndex] = blockHash
hscs.stagingAddedByHash[*blockHash] = currentIndex
currentIndex++
}
return nil
}
func (hscs *headersSelectedChainStore) IsStaged() bool {
return len(hscs.stagingAddedByHash) != 0 ||
len(hscs.stagingRemovedByHash) != 0 ||
len(hscs.stagingAddedByIndex) != 0 ||
len(hscs.stagingAddedByIndex) != 0
}
func (hscs *headersSelectedChainStore) Discard() {
hscs.stagingAddedByHash = make(map[externalapi.DomainHash]uint64)
hscs.stagingRemovedByHash = make(map[externalapi.DomainHash]struct{})
hscs.stagingAddedByIndex = make(map[uint64]*externalapi.DomainHash)
hscs.stagingRemovedByIndex = make(map[uint64]struct{})
}
func (hscs *headersSelectedChainStore) Commit(dbTx model.DBTransaction) error {
if !hscs.IsStaged() {
return nil
}
for hash := range hscs.stagingRemovedByHash {
hashCopy := hash
err := dbTx.Delete(hscs.hashAsKey(&hashCopy))
if err != nil {
return err
}
hscs.cacheByHash.Remove(&hashCopy)
}
for index := range hscs.stagingRemovedByIndex {
err := dbTx.Delete(hscs.indexAsKey(index))
if err != nil {
return err
}
hscs.cacheByIndex.Remove(index)
}
highestIndex := uint64(0)
for hash, index := range hscs.stagingAddedByHash {
hashCopy := hash
err := dbTx.Put(hscs.hashAsKey(&hashCopy), hscs.serializeIndex(index))
if err != nil {
return err
}
err = dbTx.Put(hscs.indexAsKey(index), binaryserialization.SerializeHash(&hashCopy))
if err != nil {
return err
}
hscs.cacheByHash.Add(&hashCopy, index)
hscs.cacheByIndex.Add(index, &hashCopy)
if index > highestIndex {
highestIndex = index
}
}
err := dbTx.Put(highestChainBlockIndexKey, hscs.serializeIndex(highestIndex))
if err != nil {
return err
}
hscs.cacheHighestChainBlockIndex = highestIndex
hscs.Discard()
return nil
}
// Get gets the chain block index for the given blockHash
func (hscs *headersSelectedChainStore) GetIndexByHash(dbContext model.DBReader, blockHash *externalapi.DomainHash) (uint64, error) {
if index, ok := hscs.stagingAddedByHash[*blockHash]; ok {
return index, nil
}
if _, ok := hscs.stagingRemovedByHash[*blockHash]; ok {
return 0, errors.Wrapf(database.ErrNotFound, "couldn't find block %s", blockHash)
}
if index, ok := hscs.cacheByHash.Get(blockHash); ok {
return index.(uint64), nil
}
indexBytes, err := dbContext.Get(hscs.hashAsKey(blockHash))
if err != nil {
return 0, err
}
index := hscs.deserializeIndex(indexBytes)
hscs.cacheByHash.Add(blockHash, index)
return index, nil
}
func (hscs *headersSelectedChainStore) GetHashByIndex(dbContext model.DBReader, index uint64) (*externalapi.DomainHash, error) {
if blockHash, ok := hscs.stagingAddedByIndex[index]; ok {
return blockHash, nil
}
if _, ok := hscs.stagingRemovedByIndex[index]; ok {
return nil, errors.Wrapf(database.ErrNotFound, "couldn't find chain block with index %d", index)
}
if blockHash, ok := hscs.cacheByIndex.Get(index); ok {
return blockHash, nil
}
hashBytes, err := dbContext.Get(hscs.indexAsKey(index))
if err != nil {
return nil, err
}
blockHash, err := binaryserialization.DeserializeHash(hashBytes)
if err != nil {
return nil, err
}
hscs.cacheByIndex.Add(index, blockHash)
return blockHash, nil
}
func (hscs *headersSelectedChainStore) serializeIndex(index uint64) []byte {
return binaryserialization.SerializeChainBlockIndex(index)
}
func (hscs *headersSelectedChainStore) deserializeIndex(indexBytes []byte) uint64 {
return binaryserialization.DeserializeChainBlockIndex(indexBytes)
}
func (hscs *headersSelectedChainStore) hashAsKey(hash *externalapi.DomainHash) model.DBKey {
return bucketChainBlockIndexByHash.Key(hash.ByteSlice())
}
func (hscs *headersSelectedChainStore) indexAsKey(index uint64) model.DBKey {
var keyBytes [8]byte
binary.BigEndian.PutUint64(keyBytes[:], index)
return bucketChainBlockHashByIndex.Key(keyBytes[:])
}
func (hscs *headersSelectedChainStore) highestChainBlockIndex(dbContext model.DBReader) (uint64, bool, error) {
if hscs.cacheHighestChainBlockIndex != 0 {
return hscs.cacheHighestChainBlockIndex, true, nil
}
indexBytes, err := dbContext.Get(highestChainBlockIndexKey)
if err != nil {
if errors.Is(err, database.ErrNotFound) {
return 0, false, nil
}
return 0, false, err
}
index := hscs.deserializeIndex(indexBytes)
hscs.cacheHighestChainBlockIndex = index
return index, true, nil
}

View File

@ -1,6 +1,7 @@
package consensus package consensus
import ( import (
"github.com/kaspanet/kaspad/domain/consensus/datastructures/headersselectedchainstore"
"io/ioutil" "io/ioutil"
"os" "os"
"sync" "sync"
@ -89,6 +90,7 @@ func (f *factory) NewConsensus(dagParams *dagconfig.Params, db infrastructuredat
ghostdagDataStore := ghostdagdatastore.New(pruningWindowSizeForCaches) ghostdagDataStore := ghostdagdatastore.New(pruningWindowSizeForCaches)
headersSelectedTipStore := headersselectedtipstore.New() headersSelectedTipStore := headersselectedtipstore.New()
finalityStore := finalitystore.New(200) finalityStore := finalitystore.New(200)
headersSelectedChainStore := headersselectedchainstore.New(pruningWindowSizeForCaches)
// Processes // Processes
reachabilityManager := reachabilitymanager.New( reachabilityManager := reachabilitymanager.New(
@ -148,7 +150,8 @@ func (f *factory) NewConsensus(dagParams *dagconfig.Params, db infrastructuredat
dagParams.CoinbasePayloadScriptPublicKeyMaxLength, dagParams.CoinbasePayloadScriptPublicKeyMaxLength,
ghostdagDataStore, ghostdagDataStore,
acceptanceDataStore) acceptanceDataStore)
headerTipsManager := headersselectedtipmanager.New(dbManager, dagTopologyManager, ghostdagManager, headersSelectedTipStore) headerTipsManager := headersselectedtipmanager.New(dbManager, dagTopologyManager, dagTraversalManager,
ghostdagManager, headersSelectedTipStore, headersSelectedChainStore)
genesisHash := dagParams.GenesisHash genesisHash := dagParams.GenesisHash
finalityManager := finalitymanager.New( finalityManager := finalitymanager.New(
dbManager, dbManager,
@ -258,7 +261,8 @@ func (f *factory) NewConsensus(dagParams *dagconfig.Params, db infrastructuredat
blockStatusStore, blockStatusStore,
blockHeaderStore, blockHeaderStore,
blockStore, blockStore,
pruningStore) pruningStore,
headersSelectedChainStore)
blockBuilder := blockbuilder.New( blockBuilder := blockbuilder.New(
dbManager, dbManager,
@ -300,7 +304,8 @@ func (f *factory) NewConsensus(dagParams *dagconfig.Params, db infrastructuredat
utxoDiffStore, utxoDiffStore,
blockHeaderStore, blockHeaderStore,
headersSelectedTipStore, headersSelectedTipStore,
finalityStore) finalityStore,
headersSelectedChainStore)
c := &consensus{ c := &consensus{
lock: &sync.Mutex{}, lock: &sync.Mutex{},
@ -337,6 +342,7 @@ func (f *factory) NewConsensus(dagParams *dagconfig.Params, db infrastructuredat
reachabilityDataStore: reachabilityDataStore, reachabilityDataStore: reachabilityDataStore,
utxoDiffStore: utxoDiffStore, utxoDiffStore: utxoDiffStore,
finalityStore: finalityStore, finalityStore: finalityStore,
headersSelectedChainStore: headersSelectedChainStore,
} }
genesisInfo, err := c.GetBlockInfo(genesisHash) genesisInfo, err := c.GetBlockInfo(genesisHash)

View File

@ -18,12 +18,13 @@ type Consensus interface {
ValidateAndInsertPruningPoint(newPruningPoint *DomainBlock, serializedUTXOSet []byte) error ValidateAndInsertPruningPoint(newPruningPoint *DomainBlock, serializedUTXOSet []byte) error
GetVirtualSelectedParent() (*DomainHash, error) GetVirtualSelectedParent() (*DomainHash, error)
CreateBlockLocator(lowHash, highHash *DomainHash, limit uint32) (BlockLocator, error) CreateBlockLocator(lowHash, highHash *DomainHash, limit uint32) (BlockLocator, error)
CreateHeadersSelectedChainBlockLocator(lowHash, highHash *DomainHash) (BlockLocator, error)
FindNextBlockLocatorBoundaries(blockLocator BlockLocator) (lowHash, highHash *DomainHash, err error) FindNextBlockLocatorBoundaries(blockLocator BlockLocator) (lowHash, highHash *DomainHash, err error)
GetSyncInfo() (*SyncInfo, error) GetSyncInfo() (*SyncInfo, error)
Tips() ([]*DomainHash, error) Tips() ([]*DomainHash, error)
GetVirtualInfo() (*VirtualInfo, error) GetVirtualInfo() (*VirtualInfo, error)
IsValidPruningPoint(blockHash *DomainHash) (bool, error) IsValidPruningPoint(blockHash *DomainHash) (bool, error)
GetVirtualSelectedParentChainFromBlock(blockHash *DomainHash) (*SelectedParentChainChanges, error) GetVirtualSelectedParentChainFromBlock(blockHash *DomainHash) (*SelectedChainPath, error)
IsInSelectedParentChainOf(blockHashA *DomainHash, blockHashB *DomainHash) (bool, error) IsInSelectedParentChainOf(blockHashA *DomainHash, blockHashB *DomainHash) (bool, error)
GetHeadersSelectedTip() (*DomainHash, error) GetHeadersSelectedTip() (*DomainHash, error)
} }

View File

@ -2,11 +2,11 @@ package externalapi
// BlockInsertionResult is auxiliary data returned from ValidateAndInsertBlock // BlockInsertionResult is auxiliary data returned from ValidateAndInsertBlock
type BlockInsertionResult struct { type BlockInsertionResult struct {
VirtualSelectedParentChainChanges *SelectedParentChainChanges VirtualSelectedParentChainChanges *SelectedChainPath
} }
// SelectedParentChainChanges is the set of changes made to the selected parent chain // SelectedChainPath is a path the of the selected chains between two blocks.
type SelectedParentChainChanges struct { type SelectedChainPath struct {
Added []*DomainHash Added []*DomainHash
Removed []*DomainHash Removed []*DomainHash
} }

View File

@ -0,0 +1,13 @@
package model
import "github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
// HeadersSelectedChainStore represents a store of the headers selected chain
type HeadersSelectedChainStore interface {
Store
Stage(dbContext DBReader,
chainChanges *externalapi.SelectedChainPath) error
IsStaged() bool
GetIndexByHash(dbContext DBReader, blockHash *externalapi.DomainHash) (uint64, error)
GetHashByIndex(dbContext DBReader, index uint64) (*externalapi.DomainHash, error)
}

View File

@ -4,10 +4,10 @@ import "github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
// ConsensusStateManager manages the node's consensus state // ConsensusStateManager manages the node's consensus state
type ConsensusStateManager interface { type ConsensusStateManager interface {
AddBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) AddBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error)
PopulateTransactionWithUTXOEntries(transaction *externalapi.DomainTransaction) error PopulateTransactionWithUTXOEntries(transaction *externalapi.DomainTransaction) error
UpdatePruningPoint(newPruningPoint *externalapi.DomainBlock, serializedUTXOSet []byte) error UpdatePruningPoint(newPruningPoint *externalapi.DomainBlock, serializedUTXOSet []byte) error
RestorePastUTXOSetIterator(blockHash *externalapi.DomainHash) (ReadOnlyUTXOSetIterator, error) RestorePastUTXOSetIterator(blockHash *externalapi.DomainHash) (ReadOnlyUTXOSetIterator, error)
CalculatePastUTXOAndAcceptanceData(blockHash *externalapi.DomainHash) (UTXODiff, externalapi.AcceptanceData, Multiset, error) CalculatePastUTXOAndAcceptanceData(blockHash *externalapi.DomainHash) (UTXODiff, externalapi.AcceptanceData, Multiset, error)
GetVirtualSelectedParentChainFromBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) GetVirtualSelectedParentChainFromBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error)
} }

View File

@ -13,4 +13,6 @@ type DAGTraversalManager interface {
BlueWindow(highHash *externalapi.DomainHash, windowSize int) ([]*externalapi.DomainHash, error) BlueWindow(highHash *externalapi.DomainHash, windowSize int) ([]*externalapi.DomainHash, error)
NewDownHeap() BlockHeap NewDownHeap() BlockHeap
NewUpHeap() BlockHeap NewUpHeap() BlockHeap
CalculateChainPath(
fromBlockHash, toBlockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error)
} }

View File

@ -7,6 +7,7 @@ type SyncManager interface {
GetHashesBetween(lowHash, highHash *externalapi.DomainHash, maxBlueScoreDifference uint64) ([]*externalapi.DomainHash, error) GetHashesBetween(lowHash, highHash *externalapi.DomainHash, maxBlueScoreDifference uint64) ([]*externalapi.DomainHash, error)
GetMissingBlockBodyHashes(highHash *externalapi.DomainHash) ([]*externalapi.DomainHash, error) GetMissingBlockBodyHashes(highHash *externalapi.DomainHash) ([]*externalapi.DomainHash, error)
CreateBlockLocator(lowHash, highHash *externalapi.DomainHash, limit uint32) (externalapi.BlockLocator, error) CreateBlockLocator(lowHash, highHash *externalapi.DomainHash, limit uint32) (externalapi.BlockLocator, error)
CreateHeadersSelectedChainBlockLocator(lowHash, highHash *externalapi.DomainHash) (externalapi.BlockLocator, error)
FindNextBlockLocatorBoundaries(blockLocator externalapi.BlockLocator) (lowHash, highHash *externalapi.DomainHash, err error) FindNextBlockLocatorBoundaries(blockLocator externalapi.BlockLocator) (lowHash, highHash *externalapi.DomainHash, err error)
GetSyncInfo() (*externalapi.SyncInfo, error) GetSyncInfo() (*externalapi.SyncInfo, error)
} }

View File

@ -40,6 +40,7 @@ type TestConsensus interface {
PruningStore() model.PruningStore PruningStore() model.PruningStore
ReachabilityDataStore() model.ReachabilityDataStore ReachabilityDataStore() model.ReachabilityDataStore
UTXODiffStore() model.UTXODiffStore UTXODiffStore() model.UTXODiffStore
HeadersSelectedChainStore() model.HeadersSelectedChainStore
BlockBuilder() TestBlockBuilder BlockBuilder() TestBlockBuilder
BlockProcessor() model.BlockProcessor BlockProcessor() model.BlockProcessor

View File

@ -37,6 +37,7 @@ type blockProcessor struct {
blockHeaderStore model.BlockHeaderStore blockHeaderStore model.BlockHeaderStore
headersSelectedTipStore model.HeaderSelectedTipStore headersSelectedTipStore model.HeaderSelectedTipStore
finalityStore model.FinalityStore finalityStore model.FinalityStore
headersSelectedChainStore model.HeadersSelectedChainStore
stores []model.Store stores []model.Store
} }
@ -70,6 +71,7 @@ func New(
blockHeaderStore model.BlockHeaderStore, blockHeaderStore model.BlockHeaderStore,
headersSelectedTipStore model.HeaderSelectedTipStore, headersSelectedTipStore model.HeaderSelectedTipStore,
finalityStore model.FinalityStore, finalityStore model.FinalityStore,
headersSelectedChainStore model.HeadersSelectedChainStore,
) model.BlockProcessor { ) model.BlockProcessor {
return &blockProcessor{ return &blockProcessor{
@ -100,6 +102,7 @@ func New(
blockHeaderStore: blockHeaderStore, blockHeaderStore: blockHeaderStore,
headersSelectedTipStore: headersSelectedTipStore, headersSelectedTipStore: headersSelectedTipStore,
finalityStore: finalityStore, finalityStore: finalityStore,
headersSelectedChainStore: headersSelectedChainStore,
stores: []model.Store{ stores: []model.Store{
consensusStateStore, consensusStateStore,
@ -116,6 +119,7 @@ func New(
blockHeaderStore, blockHeaderStore,
headersSelectedTipStore, headersSelectedTipStore,
finalityStore, finalityStore,
headersSelectedChainStore,
}, },
} }
} }

View File

@ -81,7 +81,7 @@ func (bp *blockProcessor) validateAndInsertBlock(block *externalapi.DomainBlock,
return nil, err return nil, err
} }
var selectedParentChainChanges *externalapi.SelectedParentChainChanges var selectedParentChainChanges *externalapi.SelectedChainPath
isHeaderOnlyBlock := isHeaderOnlyBlock(block) isHeaderOnlyBlock := isHeaderOnlyBlock(block)
if !isHeaderOnlyBlock { if !isHeaderOnlyBlock {
// There's no need to update the consensus state manager when // There's no need to update the consensus state manager when

View File

@ -9,8 +9,6 @@ import (
"github.com/pkg/errors" "github.com/pkg/errors"
) )
var byteOrder = binary.LittleEndian
const uint64Len = 8 const uint64Len = 8
const uint16Len = 2 const uint16Len = 2
const lengthOfscriptPubKeyLength = 1 const lengthOfscriptPubKeyLength = 1
@ -25,7 +23,7 @@ func (c *coinbaseManager) serializeCoinbasePayload(blueScore uint64, coinbaseDat
} }
payload := make([]byte, uint64Len+lengthOfVersionScriptPubKey+lengthOfscriptPubKeyLength+scriptLengthOfScriptPubKey+len(coinbaseData.ExtraData)) payload := make([]byte, uint64Len+lengthOfVersionScriptPubKey+lengthOfscriptPubKeyLength+scriptLengthOfScriptPubKey+len(coinbaseData.ExtraData))
byteOrder.PutUint64(payload[:uint64Len], blueScore) binary.LittleEndian.PutUint64(payload[:uint64Len], blueScore)
if len(coinbaseData.ScriptPublicKey.Script) > math.MaxUint8 { if len(coinbaseData.ScriptPublicKey.Script) > math.MaxUint8 {
return nil, errors.Errorf("script public key is bigger than %d", math.MaxUint8) return nil, errors.Errorf("script public key is bigger than %d", math.MaxUint8)
} }
@ -47,7 +45,7 @@ func (c *coinbaseManager) ExtractCoinbaseDataAndBlueScore(coinbaseTx *externalap
"coinbase payload is less than the minimum length of %d", minLength) "coinbase payload is less than the minimum length of %d", minLength)
} }
blueScore = byteOrder.Uint64(coinbaseTx.Payload[:uint64Len]) blueScore = binary.LittleEndian.Uint64(coinbaseTx.Payload[:uint64Len])
scriptPubKeyVersion := uint16(coinbaseTx.Payload[uint64Len]) scriptPubKeyVersion := uint16(coinbaseTx.Payload[uint64Len])
scriptPubKeyScriptLength := coinbaseTx.Payload[uint64Len+lengthOfVersionScriptPubKey] scriptPubKeyScriptLength := coinbaseTx.Payload[uint64Len+lengthOfVersionScriptPubKey]

View File

@ -9,7 +9,7 @@ import (
// AddBlock submits the given block to be added to the // AddBlock submits the given block to be added to the
// current virtual. This process may result in a new virtual block // current virtual. This process may result in a new virtual block
// getting created // getting created
func (csm *consensusStateManager) AddBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) { func (csm *consensusStateManager) AddBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error) {
logger.LogAndMeasureExecutionTime(log, "csm.AddBlock") logger.LogAndMeasureExecutionTime(log, "csm.AddBlock")
log.Debugf("Resolving whether the block %s is the next virtual selected parent", blockHash) log.Debugf("Resolving whether the block %s is the next virtual selected parent", blockHash)

View File

@ -6,7 +6,7 @@ import (
) )
func (csm *consensusStateManager) GetVirtualSelectedParentChainFromBlock( func (csm *consensusStateManager) GetVirtualSelectedParentChainFromBlock(
blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) { blockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error) {
// Calculate chain changes between the given blockHash and the // Calculate chain changes between the given blockHash and the
// virtual's selected parent. Note that we explicitly don't // virtual's selected parent. Note that we explicitly don't
@ -18,55 +18,5 @@ func (csm *consensusStateManager) GetVirtualSelectedParentChainFromBlock(
} }
virtualSelectedParent := virtualGHOSTDAGData.SelectedParent() virtualSelectedParent := virtualGHOSTDAGData.SelectedParent()
return csm.calculateSelectedParentChainChanges(blockHash, virtualSelectedParent) return csm.dagTraversalManager.CalculateChainPath(blockHash, virtualSelectedParent)
}
func (csm *consensusStateManager) calculateSelectedParentChainChanges(
fromBlockHash, toBlockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) {
// Walk down from fromBlockHash until we reach the common selected
// parent chain ancestor of fromBlockHash and toBlockHash. Note
// that this slice will be empty if fromBlockHash is the selected
// parent of toBlockHash
var removed []*externalapi.DomainHash
current := fromBlockHash
for {
isCurrentInTheSelectedParentChainOfNewVirtualSelectedParent, err := csm.dagTopologyManager.IsInSelectedParentChainOf(current, toBlockHash)
if err != nil {
return nil, err
}
if isCurrentInTheSelectedParentChainOfNewVirtualSelectedParent {
break
}
removed = append(removed, current)
currentGHOSTDAGData, err := csm.ghostdagDataStore.Get(csm.databaseContext, current)
if err != nil {
return nil, err
}
current = currentGHOSTDAGData.SelectedParent()
}
commonAncestor := current
// Walk down from the toBlockHash to the common ancestor
var added []*externalapi.DomainHash
current = toBlockHash
for !current.Equal(commonAncestor) {
added = append(added, current)
currentGHOSTDAGData, err := csm.ghostdagDataStore.Get(csm.databaseContext, current)
if err != nil {
return nil, err
}
current = currentGHOSTDAGData.SelectedParent()
}
// Reverse the order of `added` so that it's sorted from low hash to high hash
for i, j := 0, len(added)-1; i < j; i, j = i+1, j-1 {
added[i], added[j] = added[j], added[i]
}
return &externalapi.SelectedParentChainChanges{
Added: added,
Removed: removed,
}, nil
} }

View File

@ -10,10 +10,10 @@ import (
"github.com/kaspanet/kaspad/domain/dagconfig" "github.com/kaspanet/kaspad/domain/dagconfig"
) )
func TestCalculateSelectedParentChainChanges(t *testing.T) { func TestCalculateChainPath(t *testing.T) {
testutils.ForAllNets(t, true, func(t *testing.T, params *dagconfig.Params) { testutils.ForAllNets(t, true, func(t *testing.T, params *dagconfig.Params) {
factory := consensus.NewFactory() factory := consensus.NewFactory()
consensus, teardown, err := factory.NewTestConsensus(params, false, "TestCalculateSelectedParentChainChanges") consensus, teardown, err := factory.NewTestConsensus(params, false, "TestCalculateChainPath")
if err != nil { if err != nil {
t.Fatalf("Error setting up consensus: %+v", err) t.Fatalf("Error setting up consensus: %+v", err)
} }

View File

@ -6,7 +6,7 @@ import (
) )
func (csm *consensusStateManager) updateVirtual(newBlockHash *externalapi.DomainHash, func (csm *consensusStateManager) updateVirtual(newBlockHash *externalapi.DomainHash,
tips []*externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) { tips []*externalapi.DomainHash) (*externalapi.SelectedChainPath, error) {
log.Debugf("updateVirtual start for block %s", newBlockHash) log.Debugf("updateVirtual start for block %s", newBlockHash)
defer log.Debugf("updateVirtual end for block %s", newBlockHash) defer log.Debugf("updateVirtual end for block %s", newBlockHash)
@ -64,14 +64,15 @@ func (csm *consensusStateManager) updateVirtual(newBlockHash *externalapi.Domain
} }
log.Debugf("Calculating selected parent chain changes") log.Debugf("Calculating selected parent chain changes")
var selectedParentChainChanges *externalapi.SelectedParentChainChanges var selectedParentChainChanges *externalapi.SelectedChainPath
if !newBlockHash.Equal(csm.genesisHash) { if !newBlockHash.Equal(csm.genesisHash) {
newVirtualGHOSTDAGData, err := csm.ghostdagDataStore.Get(csm.databaseContext, model.VirtualBlockHash) newVirtualGHOSTDAGData, err := csm.ghostdagDataStore.Get(csm.databaseContext, model.VirtualBlockHash)
if err != nil { if err != nil {
return nil, err return nil, err
} }
newVirtualSelectedParent := newVirtualGHOSTDAGData.SelectedParent() newVirtualSelectedParent := newVirtualGHOSTDAGData.SelectedParent()
selectedParentChainChanges, err = csm.calculateSelectedParentChainChanges(oldVirtualSelectedParent, newVirtualSelectedParent) selectedParentChainChanges, err = csm.dagTraversalManager.
CalculateChainPath(oldVirtualSelectedParent, newVirtualSelectedParent)
if err != nil { if err != nil {
return nil, err return nil, err
} }

View File

@ -145,3 +145,53 @@ func (dtm *dagTraversalManager) LowestChainBlockAboveOrEqualToBlueScore(highHash
return currentHash, nil return currentHash, nil
} }
func (dtm *dagTraversalManager) CalculateChainPath(
fromBlockHash, toBlockHash *externalapi.DomainHash) (*externalapi.SelectedChainPath, error) {
// Walk down from fromBlockHash until we reach the common selected
// parent chain ancestor of fromBlockHash and toBlockHash. Note
// that this slice will be empty if fromBlockHash is the selected
// parent of toBlockHash
var removed []*externalapi.DomainHash
current := fromBlockHash
for {
isCurrentInTheSelectedParentChainOfNewVirtualSelectedParent, err := dtm.dagTopologyManager.IsInSelectedParentChainOf(current, toBlockHash)
if err != nil {
return nil, err
}
if isCurrentInTheSelectedParentChainOfNewVirtualSelectedParent {
break
}
removed = append(removed, current)
currentGHOSTDAGData, err := dtm.ghostdagDataStore.Get(dtm.databaseContext, current)
if err != nil {
return nil, err
}
current = currentGHOSTDAGData.SelectedParent()
}
commonAncestor := current
// Walk down from the toBlockHash to the common ancestor
var added []*externalapi.DomainHash
current = toBlockHash
for !current.Equal(commonAncestor) {
added = append(added, current)
currentGHOSTDAGData, err := dtm.ghostdagDataStore.Get(dtm.databaseContext, current)
if err != nil {
return nil, err
}
current = currentGHOSTDAGData.SelectedParent()
}
// Reverse the order of `added` so that it's sorted from low hash to high hash
for i, j := 0, len(added)-1; i < j; i, j = i+1, j-1 {
added[i], added[j] = added[j], added[i]
}
return &externalapi.SelectedChainPath{
Added: added,
Removed: removed,
}, nil
}

View File

@ -0,0 +1,79 @@
package headersselectedtipmanager
import (
"github.com/kaspanet/kaspad/domain/consensus/model"
"github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
)
type headerTipsManager struct {
databaseContext model.DBReader
dagTopologyManager model.DAGTopologyManager
dagTraversalManager model.DAGTraversalManager
ghostdagManager model.GHOSTDAGManager
headersSelectedTipStore model.HeaderSelectedTipStore
headersSelectedChainStore model.HeadersSelectedChainStore
}
// New instantiates a new HeadersSelectedTipManager
func New(databaseContext model.DBReader,
dagTopologyManager model.DAGTopologyManager,
dagTraversalManager model.DAGTraversalManager,
ghostdagManager model.GHOSTDAGManager,
headersSelectedTipStore model.HeaderSelectedTipStore,
headersSelectedChainStore model.HeadersSelectedChainStore) model.HeadersSelectedTipManager {
return &headerTipsManager{
databaseContext: databaseContext,
dagTopologyManager: dagTopologyManager,
dagTraversalManager: dagTraversalManager,
ghostdagManager: ghostdagManager,
headersSelectedTipStore: headersSelectedTipStore,
headersSelectedChainStore: headersSelectedChainStore,
}
}
func (h *headerTipsManager) AddHeaderTip(hash *externalapi.DomainHash) error {
hasSelectedTip, err := h.headersSelectedTipStore.Has(h.databaseContext)
if err != nil {
return err
}
if !hasSelectedTip {
h.headersSelectedTipStore.Stage(hash)
err := h.headersSelectedChainStore.Stage(h.databaseContext, &externalapi.SelectedChainPath{
Added: []*externalapi.DomainHash{hash},
Removed: nil,
})
if err != nil {
return err
}
} else {
headersSelectedTip, err := h.headersSelectedTipStore.HeadersSelectedTip(h.databaseContext)
if err != nil {
return err
}
newHeadersSelectedTip, err := h.ghostdagManager.ChooseSelectedParent(headersSelectedTip, hash)
if err != nil {
return err
}
if !newHeadersSelectedTip.Equal(headersSelectedTip) {
h.headersSelectedTipStore.Stage(newHeadersSelectedTip)
chainChanges, err := h.dagTraversalManager.CalculateChainPath(headersSelectedTip, newHeadersSelectedTip)
if err != nil {
return err
}
err = h.headersSelectedChainStore.Stage(h.databaseContext, chainChanges)
if err != nil {
return err
}
}
}
return nil
}

View File

@ -0,0 +1,77 @@
package headersselectedtipmanager_test
import (
"github.com/kaspanet/kaspad/domain/consensus"
"github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
"github.com/kaspanet/kaspad/domain/consensus/utils/testutils"
"github.com/kaspanet/kaspad/domain/dagconfig"
"github.com/kaspanet/kaspad/infrastructure/db/database"
"github.com/pkg/errors"
"testing"
)
func TestAddHeaderTip(t *testing.T) {
testutils.ForAllNets(t, true, func(t *testing.T, params *dagconfig.Params) {
factory := consensus.NewFactory()
tc, tearDown, err := factory.NewTestConsensus(params, false, "TestAddHeaderTip")
if err != nil {
t.Fatalf("NewTestConsensus: %s", err)
}
defer tearDown(false)
checkExpectedSelectedChain := func(expectedSelectedChain []*externalapi.DomainHash) {
for i, blockHash := range expectedSelectedChain {
chainBlockHash, err := tc.HeadersSelectedChainStore().GetHashByIndex(tc.DatabaseContext(), uint64(i))
if err != nil {
t.Fatalf("GetHashByIndex: %+v", err)
}
if !blockHash.Equal(chainBlockHash) {
t.Fatalf("chain block %d is expected to be %s but got %s", i, blockHash, chainBlockHash)
}
index, err := tc.HeadersSelectedChainStore().GetIndexByHash(tc.DatabaseContext(), blockHash)
if err != nil {
t.Fatalf("GetIndexByHash: %+v", err)
}
if uint64(i) != index {
t.Fatalf("chain block %s is expected to be %d but got %d", blockHash, i, index)
}
}
_, err := tc.HeadersSelectedChainStore().GetHashByIndex(tc.DatabaseContext(),
uint64(len(expectedSelectedChain)+1))
if !errors.Is(err, database.ErrNotFound) {
t.Fatalf("index %d is not expected to exist, but instead got error: %+v",
uint64(len(expectedSelectedChain)+1), err)
}
}
expectedSelectedChain := []*externalapi.DomainHash{params.GenesisHash}
tipHash := params.GenesisHash
for i := 0; i < 10; i++ {
var err error
tipHash, _, err = tc.AddBlock([]*externalapi.DomainHash{tipHash}, nil, nil)
if err != nil {
t.Fatalf("AddBlock: %+v", err)
}
expectedSelectedChain = append(expectedSelectedChain, tipHash)
checkExpectedSelectedChain(expectedSelectedChain)
}
expectedSelectedChain = []*externalapi.DomainHash{params.GenesisHash}
tipHash = params.GenesisHash
for i := 0; i < 11; i++ {
var err error
tipHash, _, err = tc.AddBlock([]*externalapi.DomainHash{tipHash}, nil, nil)
if err != nil {
t.Fatalf("AddBlock: %+v", err)
}
expectedSelectedChain = append(expectedSelectedChain, tipHash)
}
checkExpectedSelectedChain(expectedSelectedChain)
})
}

View File

@ -1,55 +0,0 @@
package headersselectedtipmanager
import (
"github.com/kaspanet/kaspad/domain/consensus/model"
"github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
)
type headerTipsManager struct {
databaseContext model.DBReader
dagTopologyManager model.DAGTopologyManager
ghostdagManager model.GHOSTDAGManager
headersSelectedTipStore model.HeaderSelectedTipStore
}
// New instantiates a new HeadersSelectedTipManager
func New(databaseContext model.DBReader,
dagTopologyManager model.DAGTopologyManager,
ghostdagManager model.GHOSTDAGManager,
headersSelectedTipStore model.HeaderSelectedTipStore) model.HeadersSelectedTipManager {
return &headerTipsManager{
databaseContext: databaseContext,
dagTopologyManager: dagTopologyManager,
ghostdagManager: ghostdagManager,
headersSelectedTipStore: headersSelectedTipStore,
}
}
func (h *headerTipsManager) AddHeaderTip(hash *externalapi.DomainHash) error {
hasSelectedTip, err := h.headersSelectedTipStore.Has(h.databaseContext)
if err != nil {
return err
}
if !hasSelectedTip {
h.headersSelectedTipStore.Stage(hash)
} else {
headersSelectedTip, err := h.headersSelectedTipStore.HeadersSelectedTip(h.databaseContext)
if err != nil {
return err
}
newHeadersSelectedTip, err := h.ghostdagManager.ChooseSelectedParent(headersSelectedTip, hash)
if err != nil {
return err
}
if !newHeadersSelectedTip.Equal(headersSelectedTip) {
h.headersSelectedTipStore.Stage(newHeadersSelectedTip)
}
}
return nil
}

View File

@ -98,3 +98,46 @@ func (sm *syncManager) findNextBlockLocatorBoundaries(blockLocator externalapi.B
} }
return highestKnownHash, lowestUnknownHash, nil return highestKnownHash, lowestUnknownHash, nil
} }
func (sm *syncManager) createHeadersSelectedChainBlockLocator(lowHash,
highHash *externalapi.DomainHash) (externalapi.BlockLocator, error) {
if highHash.Equal(sm.genesisBlockHash) && lowHash.Equal(sm.genesisBlockHash) {
return externalapi.BlockLocator{sm.genesisBlockHash}, nil
}
lowHashIndex, err := sm.headersSelectedChainStore.GetIndexByHash(sm.databaseContext, lowHash)
if err != nil {
return nil, err
}
highHashIndex, err := sm.headersSelectedChainStore.GetIndexByHash(sm.databaseContext, highHash)
if err != nil {
return nil, err
}
if highHashIndex < lowHashIndex {
return nil, errors.Errorf("cannot build block locator while highHash is lower than lowHash")
}
locator := externalapi.BlockLocator{}
currentIndex := highHashIndex
step := uint64(1)
for currentIndex > lowHashIndex {
blockHash, err := sm.headersSelectedChainStore.GetHashByIndex(sm.databaseContext, currentIndex)
if err != nil {
return nil, err
}
locator = append(locator, blockHash)
if currentIndex < step {
break
}
currentIndex -= step
step *= 2
}
locator = append(locator, lowHash)
return locator, nil
}

View File

@ -20,6 +20,7 @@ type syncManager struct {
blockHeaderStore model.BlockHeaderStore blockHeaderStore model.BlockHeaderStore
blockStore model.BlockStore blockStore model.BlockStore
pruningStore model.PruningStore pruningStore model.PruningStore
headersSelectedChainStore model.HeadersSelectedChainStore
} }
// New instantiates a new SyncManager // New instantiates a new SyncManager
@ -35,7 +36,8 @@ func New(
blockStatusStore model.BlockStatusStore, blockStatusStore model.BlockStatusStore,
blockHeaderStore model.BlockHeaderStore, blockHeaderStore model.BlockHeaderStore,
blockStore model.BlockStore, blockStore model.BlockStore,
pruningStore model.PruningStore) model.SyncManager { pruningStore model.PruningStore,
headersSelectedChainStore model.HeadersSelectedChainStore) model.SyncManager {
return &syncManager{ return &syncManager{
databaseContext: databaseContext, databaseContext: databaseContext,
@ -45,6 +47,7 @@ func New(
dagTopologyManager: dagTopologyManager, dagTopologyManager: dagTopologyManager,
ghostdagManager: ghostdagManager, ghostdagManager: ghostdagManager,
pruningManager: pruningManager, pruningManager: pruningManager,
headersSelectedChainStore: headersSelectedChainStore,
ghostdagDataStore: ghostdagDataStore, ghostdagDataStore: ghostdagDataStore,
blockStatusStore: blockStatusStore, blockStatusStore: blockStatusStore,
@ -77,6 +80,15 @@ func (sm *syncManager) CreateBlockLocator(lowHash, highHash *externalapi.DomainH
return sm.createBlockLocator(lowHash, highHash, limit) return sm.createBlockLocator(lowHash, highHash, limit)
} }
func (sm *syncManager) CreateHeadersSelectedChainBlockLocator(lowHash,
highHash *externalapi.DomainHash) (externalapi.BlockLocator, error) {
onEnd := logger.LogAndMeasureExecutionTime(log, "CreateHeadersSelectedChainBlockLocator")
defer onEnd()
return sm.createHeadersSelectedChainBlockLocator(lowHash, highHash)
}
func (sm *syncManager) FindNextBlockLocatorBoundaries(blockLocator externalapi.BlockLocator) (lowHash, highHash *externalapi.DomainHash, err error) { func (sm *syncManager) FindNextBlockLocatorBoundaries(blockLocator externalapi.BlockLocator) (lowHash, highHash *externalapi.DomainHash, err error) {
onEnd := logger.LogAndMeasureExecutionTime(log, "FindNextBlockLocatorBoundaries") onEnd := logger.LogAndMeasureExecutionTime(log, "FindNextBlockLocatorBoundaries")
defer onEnd() defer onEnd()

View File

@ -95,4 +95,5 @@ func (tc *testConsensus) DiscardAllStores() {
tc.PruningStore().Discard() tc.PruningStore().Discard()
tc.ReachabilityDataStore().Discard() tc.ReachabilityDataStore().Discard()
tc.UTXODiffStore().Discard() tc.UTXODiffStore().Discard()
tc.HeadersSelectedChainStore().Discard()
} }

View File

@ -133,3 +133,7 @@ func (tc *testConsensus) FinalityManager() model.FinalityManager {
func (tc *testConsensus) FinalityStore() model.FinalityStore { func (tc *testConsensus) FinalityStore() model.FinalityStore {
return tc.finalityStore return tc.finalityStore
} }
func (tc *testConsensus) HeadersSelectedChainStore() model.HeadersSelectedChainStore {
return tc.headersSelectedChainStore
}

View File

@ -0,0 +1,57 @@
package lrucacheuint64tohash
import "github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
// LRUCache is a least-recently-used cache from
// uint64 to DomainHash
type LRUCache struct {
cache map[uint64]*externalapi.DomainHash
capacity int
}
// New creates a new LRUCache
func New(capacity int) *LRUCache {
return &LRUCache{
cache: make(map[uint64]*externalapi.DomainHash, capacity+1),
capacity: capacity,
}
}
// Add adds an entry to the LRUCache
func (c *LRUCache) Add(key uint64, value *externalapi.DomainHash) {
c.cache[key] = value
if len(c.cache) > c.capacity {
c.evictRandom()
}
}
// Get returns the entry for the given key, or (nil, false) otherwise
func (c *LRUCache) Get(key uint64) (*externalapi.DomainHash, bool) {
value, ok := c.cache[key]
if !ok {
return nil, false
}
return value, true
}
// Has returns whether the LRUCache contains the given key
func (c *LRUCache) Has(key uint64) bool {
_, ok := c.cache[key]
return ok
}
// Remove removes the entry for the the given key. Does nothing if
// the entry does not exist
func (c *LRUCache) Remove(key uint64) {
delete(c.cache, key)
}
func (c *LRUCache) evictRandom() {
var keyToEvict uint64
for key := range c.cache {
keyToEvict = key
break
}
c.Remove(keyToEvict)
}

View File

@ -29,7 +29,7 @@ func New(consensus externalapi.Consensus, database database.Database) *UTXOIndex
} }
// Update updates the UTXO index with the given DAG selected parent chain changes // Update updates the UTXO index with the given DAG selected parent chain changes
func (ui *UTXOIndex) Update(chainChanges *externalapi.SelectedParentChainChanges) (*UTXOChanges, error) { func (ui *UTXOIndex) Update(chainChanges *externalapi.SelectedChainPath) (*UTXOChanges, error) {
onEnd := logger.LogAndMeasureExecutionTime(log, "UTXOIndex.Update") onEnd := logger.LogAndMeasureExecutionTime(log, "UTXOIndex.Update")
defer onEnd() defer onEnd()