Compare commits

...

6 Commits

Author SHA1 Message Date
Ori Newman
ce85b9cc09 Make the pruning point the earliest chain block with finality interval higher than the previous pruning point 2020-12-21 18:33:52 +02:00
Ori Newman
f0c7e03ece Regenerate messages.pb.go 2020-12-21 15:08:19 +02:00
Ori Newman
863b9d9e07 Merge remote-tracking branch 'origin/v0.8.4-dev' into pruning-calculation-changes 2020-12-21 15:07:46 +02:00
Ori Newman
8e188adbc2 Fix UpdatePruningPointByVirtual and IsValidPruningPoint 2020-12-21 15:04:22 +02:00
Ori Newman
c27da560c5 Add request IBD root hash flow 2020-12-20 19:11:10 +02:00
Ori Newman
23f093ca17 1) Calculate pruning point incrementally
2) Add IsValidPruningPoint to pruning manager and consensus
3) Use reachability children for selected child iterator
2020-12-20 18:58:28 +02:00
28 changed files with 11384 additions and 1841 deletions

View File

@ -55,6 +55,8 @@ const (
CmdIBDRootUTXOSetAndBlock CmdIBDRootUTXOSetAndBlock
CmdRequestIBDBlocks CmdRequestIBDBlocks
CmdIBDRootNotFound CmdIBDRootNotFound
CmdRequestIBDRootHash
CmdIBDRootHash
// rpc // rpc
CmdGetCurrentNetworkRequestMessage CmdGetCurrentNetworkRequestMessage
@ -145,6 +147,8 @@ var ProtocolMessageCommandToString = map[MessageCommand]string{
CmdIBDRootUTXOSetAndBlock: "IBDRootUTXOSetAndBlock", CmdIBDRootUTXOSetAndBlock: "IBDRootUTXOSetAndBlock",
CmdRequestIBDBlocks: "RequestIBDBlocks", CmdRequestIBDBlocks: "RequestIBDBlocks",
CmdIBDRootNotFound: "IBDRootNotFound", CmdIBDRootNotFound: "IBDRootNotFound",
CmdRequestIBDRootHash: "IBDRequestIBDRootHash",
CmdIBDRootHash: "IBDIBDRootHash",
} }
// RPCMessageCommandToString maps all MessageCommands to their string representation // RPCMessageCommandToString maps all MessageCommands to their string representation

View File

@ -17,6 +17,6 @@ func (msg *MsgIBDRootNotFound) Command() MessageCommand {
// NewMsgIBDRootNotFound returns a new kaspa IBDRootNotFound message that conforms to the // NewMsgIBDRootNotFound returns a new kaspa IBDRootNotFound message that conforms to the
// Message interface. // Message interface.
func NewMsgIBDRootNotFound() *MsgDoneHeaders { func NewMsgIBDRootNotFound() *MsgIBDRootNotFound {
return &MsgDoneHeaders{} return &MsgIBDRootNotFound{}
} }

View File

@ -0,0 +1,26 @@
package appmessage
import (
"github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
)
// MsgIBDRootHash implements the Message interface and represents a kaspa
// IBDRootHash message. It is used as a reply to IBD root hash requests.
type MsgIBDRootHash struct {
baseMessage
Hash *externalapi.DomainHash
}
// Command returns the protocol command string for the message. This is part
// of the Message interface implementation.
func (msg *MsgIBDRootHash) Command() MessageCommand {
return CmdIBDRootHash
}
// NewMsgIBDRootHash returns a new kaspa IBDRootHash message that conforms to
// the Message interface. See MsgIBDRootHash for details.
func NewMsgIBDRootHash(hash *externalapi.DomainHash) *MsgIBDRootHash {
return &MsgIBDRootHash{
Hash: hash,
}
}

View File

@ -0,0 +1,22 @@
package appmessage
// MsgRequestIBDRootHash implements the Message interface and represents a kaspa
// MsgRequestIBDRootHash message. It is used to request the IBD root hash
// from a peer during IBD.
//
// This message has no payload.
type MsgRequestIBDRootHash struct {
baseMessage
}
// Command returns the protocol command string for the message. This is part
// of the Message interface implementation.
func (msg *MsgRequestIBDRootHash) Command() MessageCommand {
return CmdRequestIBDRootHash
}
// NewMsgRequestIBDRootHash returns a new kaspa RequestIBDRootHash message that conforms to the
// Message interface.
func NewMsgRequestIBDRootHash() *MsgRequestIBDRootHash {
return &MsgRequestIBDRootHash{}
}

View File

@ -0,0 +1,49 @@
package blockrelay
import (
"github.com/kaspanet/kaspad/app/appmessage"
"github.com/kaspanet/kaspad/domain"
"github.com/kaspanet/kaspad/infrastructure/network/netadapter/router"
)
// HandleIBDRootHashRequestsFlowContext is the interface for the context needed for the handleIBDRootHashRequestsFlow flow.
type HandleIBDRootHashRequestsFlowContext interface {
Domain() domain.Domain
}
type handleIBDRootHashRequestsFlow struct {
HandleIBDRootHashRequestsFlowContext
incomingRoute, outgoingRoute *router.Route
}
// HandleIBDRootHashRequests listens to appmessage.MsgRequestIBDRootHash messages and sends
// the IBD root hash as response.
func HandleIBDRootHashRequests(context HandleIBDRootHashRequestsFlowContext, incomingRoute,
outgoingRoute *router.Route) error {
flow := &handleIBDRootHashRequestsFlow{
HandleIBDRootHashRequestsFlowContext: context,
incomingRoute: incomingRoute,
outgoingRoute: outgoingRoute,
}
return flow.start()
}
func (flow *handleIBDRootHashRequestsFlow) start() error {
for {
_, err := flow.incomingRoute.Dequeue()
if err != nil {
return err
}
pruningPoint, err := flow.Domain().Consensus().PruningPoint()
if err != nil {
return err
}
err = flow.outgoingRoute.Enqueue(appmessage.NewMsgIBDRootHash(pruningPoint))
if err != nil {
return err
}
}
}

View File

@ -30,22 +30,55 @@ func (flow *handleRelayInvsFlow) runIBDIfNotRunning(highHash *externalapi.Domain
log.Debugf("Finished downloading headers up to %s", highHash) log.Debugf("Finished downloading headers up to %s", highHash)
// Fetch the UTXO set if we don't already have it // Fetch the UTXO set if we don't already have it
log.Debugf("Downloading the IBD root UTXO set under highHash %s", highHash) log.Debugf("Checking if there's a new pruning point under %s", highHash)
syncInfo, err := flow.Domain().Consensus().GetSyncInfo() err = flow.outgoingRoute.Enqueue(appmessage.NewMsgRequestIBDRootHash())
if err != nil { if err != nil {
return err return err
} }
if syncInfo.IsAwaitingUTXOSet {
found, err := flow.fetchMissingUTXOSet(syncInfo.IBDRootUTXOBlockHash) message, err := flow.dequeueIncomingMessageAndSkipInvs(common.DefaultTimeout)
if err != nil {
return err
}
msgIBDRootHash, ok := message.(*appmessage.MsgIBDRootHash)
if !ok {
return protocolerrors.Errorf(true, "received unexpected message type. "+
"expected: %s, got: %s", appmessage.CmdIBDRootHash, message.Command())
}
blockInfo, err := flow.Domain().Consensus().GetBlockInfo(msgIBDRootHash.Hash)
if err != nil {
return err
}
if blockInfo.BlockStatus == externalapi.StatusHeaderOnly {
isValid, err := flow.Domain().Consensus().IsValidPruningPoint(msgIBDRootHash.Hash)
if err != nil { if err != nil {
return err return err
} }
if !found {
log.Infof("Cannot download the IBD root UTXO set under highHash %s", highHash) if !isValid {
log.Infof("The suggested pruning point is incompatible to this node DAG, so stopping IBD with this" +
" peer")
return nil return nil
} }
log.Info("Fetching the pruning point UTXO set")
succeed, err := flow.fetchMissingUTXOSet(msgIBDRootHash.Hash)
if err != nil {
return err
}
if !succeed {
log.Infof("Couldn't successfully fetch the pruning point UTXO set. Stopping IBD.")
return nil
}
log.Info("Fetched the new pruning point UTXO set")
} else {
log.Debugf("Already has the block data of the new suggested pruning point %s", msgIBDRootHash.Hash)
} }
log.Debugf("Finished downloading the IBD root UTXO set under highHash %s", highHash)
// Fetch the block bodies // Fetch the block bodies
log.Debugf("Downloading block bodies up to %s", highHash) log.Debugf("Downloading block bodies up to %s", highHash)
@ -58,6 +91,40 @@ func (flow *handleRelayInvsFlow) runIBDIfNotRunning(highHash *externalapi.Domain
return nil return nil
} }
func (flow *handleRelayInvsFlow) fetchUTXOSetIfMissing() (bool, error) {
err := flow.outgoingRoute.Enqueue(appmessage.NewMsgRequestIBDRootHash())
if err != nil {
return false, err
}
message, err := flow.dequeueIncomingMessageAndSkipInvs(common.DefaultTimeout)
if err != nil {
return false, err
}
msgIBDRootHash, ok := message.(*appmessage.MsgIBDRootHash)
if !ok {
return false, protocolerrors.Errorf(true, "received unexpected message type. "+
"expected: %s, got: %s", appmessage.CmdIBDRootHash, message.Command())
}
isValid, err := flow.Domain().Consensus().IsValidPruningPoint(msgIBDRootHash.Hash)
if err != nil {
return false, err
}
if !isValid {
return false, nil
}
found, err := flow.fetchMissingUTXOSet(msgIBDRootHash.Hash)
if err != nil {
return false, err
}
return found, nil
}
func (flow *handleRelayInvsFlow) syncHeaders(highHash *externalapi.DomainHash) error { func (flow *handleRelayInvsFlow) syncHeaders(highHash *externalapi.DomainHash) error {
log.Debugf("Trying to find highest shared chain block with peer %s with high hash %s", flow.peer, highHash) log.Debugf("Trying to find highest shared chain block with peer %s with high hash %s", flow.peer, highHash)
highestSharedBlockHash, err := flow.findHighestSharedBlockHash(highHash) highestSharedBlockHash, err := flow.findHighestSharedBlockHash(highHash)
@ -207,21 +274,13 @@ func (flow *handleRelayInvsFlow) fetchMissingUTXOSet(ibdRootHash *externalapi.Do
err = flow.Domain().Consensus().ValidateAndInsertPruningPoint(block, utxoSet) err = flow.Domain().Consensus().ValidateAndInsertPruningPoint(block, utxoSet)
if err != nil { if err != nil {
// TODO: Find a better way to deal with finality conflicts.
if errors.Is(err, ruleerrors.ErrSuggestedPruningViolatesFinality) {
return false, nil
}
return false, protocolerrors.ConvertToBanningProtocolErrorIfRuleError(err, "error with IBD root UTXO set") return false, protocolerrors.ConvertToBanningProtocolErrorIfRuleError(err, "error with IBD root UTXO set")
} }
syncInfo, err := flow.Domain().Consensus().GetSyncInfo()
if err != nil {
return false, err
}
// TODO: Find a better way to deal with finality conflicts.
if syncInfo.IsAwaitingUTXOSet {
log.Warnf("Still awaiting for UTXO set. This can happen only because the given pruning point violates " +
"finality. If this keeps happening delete the data directory and restart your node.")
return false, nil
}
return true, nil return true, nil
} }

View File

@ -136,7 +136,7 @@ func (m *Manager) registerBlockRelayFlows(router *routerpkg.Router, isStopping *
m.registerFlow("HandleRelayInvs", router, []appmessage.MessageCommand{ m.registerFlow("HandleRelayInvs", router, []appmessage.MessageCommand{
appmessage.CmdInvRelayBlock, appmessage.CmdBlock, appmessage.CmdBlockLocator, appmessage.CmdIBDBlock, appmessage.CmdInvRelayBlock, appmessage.CmdBlock, appmessage.CmdBlockLocator, appmessage.CmdIBDBlock,
appmessage.CmdDoneHeaders, appmessage.CmdIBDRootNotFound, appmessage.CmdIBDRootUTXOSetAndBlock, appmessage.CmdDoneHeaders, appmessage.CmdIBDRootNotFound, appmessage.CmdIBDRootUTXOSetAndBlock,
appmessage.CmdHeader}, isStopping, errChan, appmessage.CmdHeader, appmessage.CmdIBDRootHash}, isStopping, errChan,
func(incomingRoute *routerpkg.Route, peer *peerpkg.Peer) error { func(incomingRoute *routerpkg.Route, peer *peerpkg.Peer) error {
return blockrelay.HandleRelayInvs(m.context, incomingRoute, return blockrelay.HandleRelayInvs(m.context, incomingRoute,
outgoingRoute, peer) outgoingRoute, peer)
@ -176,6 +176,13 @@ func (m *Manager) registerBlockRelayFlows(router *routerpkg.Router, isStopping *
return blockrelay.HandleIBDBlockRequests(m.context, incomingRoute, outgoingRoute) return blockrelay.HandleIBDBlockRequests(m.context, incomingRoute, outgoingRoute)
}, },
), ),
m.registerFlow("HandleIBDRootHashRequests", router,
[]appmessage.MessageCommand{appmessage.CmdRequestIBDRootHash}, isStopping, errChan,
func(incomingRoute *routerpkg.Route, peer *peerpkg.Peer) error {
return blockrelay.HandleIBDRootHashRequests(m.context, incomingRoute, outgoingRoute)
},
),
} }
} }

View File

@ -184,6 +184,13 @@ func (s *consensus) GetPruningPointUTXOSet(expectedPruningPointHash *externalapi
return serializedUTXOSet, nil return serializedUTXOSet, nil
} }
func (s *consensus) PruningPoint() (*externalapi.DomainHash, error) {
s.lock.Lock()
defer s.lock.Unlock()
return s.pruningStore.PruningPoint(s.databaseContext)
}
func (s *consensus) ValidateAndInsertPruningPoint(newPruningPoint *externalapi.DomainBlock, serializedUTXOSet []byte) error { func (s *consensus) ValidateAndInsertPruningPoint(newPruningPoint *externalapi.DomainBlock, serializedUTXOSet []byte) error {
s.lock.Lock() s.lock.Lock()
defer s.lock.Unlock() defer s.lock.Unlock()
@ -259,6 +266,10 @@ func (s *consensus) GetSyncInfo() (*externalapi.SyncInfo, error) {
return s.syncManager.GetSyncInfo() return s.syncManager.GetSyncInfo()
} }
func (s *consensus) IsValidPruningPoint(block *externalapi.DomainHash) (bool, error) {
return s.pruningManager.IsValidPruningPoint(block)
}
func (s *consensus) GetVirtualSelectedParentChainFromBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) { func (s *consensus) GetVirtualSelectedParentChainFromBlock(blockHash *externalapi.DomainHash) (*externalapi.SelectedParentChainChanges, error) {
s.lock.Lock() s.lock.Lock()
defer s.lock.Unlock() defer s.lock.Unlock()

View File

@ -9,13 +9,54 @@ import (
) )
var pruningBlockHashKey = dbkeys.MakeBucket().Key([]byte("pruning-block-hash")) var pruningBlockHashKey = dbkeys.MakeBucket().Key([]byte("pruning-block-hash"))
var pruningSerializedUTXOSetkey = dbkeys.MakeBucket().Key([]byte("pruning-utxo-set")) var candidatePruningPointHashKey = dbkeys.MakeBucket().Key([]byte("candidate-pruning-point-hash"))
var pruningSerializedUTXOSetKey = dbkeys.MakeBucket().Key([]byte("pruning-utxo-set"))
// pruningStore represents a store for the current pruning state // pruningStore represents a store for the current pruning state
type pruningStore struct { type pruningStore struct {
pruningPointStaging *externalapi.DomainHash pruningPointStaging *externalapi.DomainHash
serializedUTXOSetStaging []byte pruningPointCandidateStaging *externalapi.DomainHash
pruningPointCache *externalapi.DomainHash serializedUTXOSetStaging []byte
pruningPointCandidateCache *externalapi.DomainHash
pruningPointCache *externalapi.DomainHash
}
func (ps *pruningStore) StagePruningPointCandidate(candidate *externalapi.DomainHash) {
ps.pruningPointCandidateStaging = candidate.Clone()
}
func (ps *pruningStore) PruningPointCandidate(dbContext model.DBReader) (*externalapi.DomainHash, error) {
if ps.pruningPointCandidateStaging != nil {
return ps.pruningPointCandidateStaging, nil
}
if ps.pruningPointCandidateCache != nil {
return ps.pruningPointCandidateCache, nil
}
candidateBytes, err := dbContext.Get(pruningBlockHashKey)
if err != nil {
return nil, err
}
candidate, err := ps.deserializePruningPoint(candidateBytes)
if err != nil {
return nil, err
}
ps.pruningPointCandidateCache = candidate
return candidate, nil
}
func (ps *pruningStore) HasPruningPointCandidate(dbContext model.DBReader) (bool, error) {
if ps.pruningPointCandidateStaging != nil {
return true, nil
}
if ps.pruningPointCandidateCache != nil {
return true, nil
}
return dbContext.Has(candidatePruningPointHashKey)
} }
// New instantiates a new PruningStore // New instantiates a new PruningStore
@ -24,7 +65,7 @@ func New() model.PruningStore {
} }
// Stage stages the pruning state // Stage stages the pruning state
func (ps *pruningStore) Stage(pruningPointBlockHash *externalapi.DomainHash, pruningPointUTXOSetBytes []byte) { func (ps *pruningStore) StagePruningPoint(pruningPointBlockHash *externalapi.DomainHash, pruningPointUTXOSetBytes []byte) {
ps.pruningPointStaging = pruningPointBlockHash.Clone() ps.pruningPointStaging = pruningPointBlockHash.Clone()
ps.serializedUTXOSetStaging = pruningPointUTXOSetBytes ps.serializedUTXOSetStaging = pruningPointUTXOSetBytes
} }
@ -40,7 +81,7 @@ func (ps *pruningStore) Discard() {
func (ps *pruningStore) Commit(dbTx model.DBTransaction) error { func (ps *pruningStore) Commit(dbTx model.DBTransaction) error {
if ps.pruningPointStaging != nil { if ps.pruningPointStaging != nil {
pruningPointBytes, err := ps.serializePruningPoint(ps.pruningPointStaging) pruningPointBytes, err := ps.serializeHash(ps.pruningPointStaging)
if err != nil { if err != nil {
return err return err
} }
@ -51,12 +92,24 @@ func (ps *pruningStore) Commit(dbTx model.DBTransaction) error {
ps.pruningPointCache = ps.pruningPointStaging ps.pruningPointCache = ps.pruningPointStaging
} }
if ps.pruningPointCandidateStaging != nil {
candidateBytes, err := ps.serializeHash(ps.pruningPointCandidateStaging)
if err != nil {
return err
}
err = dbTx.Put(candidatePruningPointHashKey, candidateBytes)
if err != nil {
return err
}
ps.pruningPointCandidateCache = ps.pruningPointCandidateStaging
}
if ps.serializedUTXOSetStaging != nil { if ps.serializedUTXOSetStaging != nil {
utxoSetBytes, err := ps.serializeUTXOSetBytes(ps.serializedUTXOSetStaging) utxoSetBytes, err := ps.serializeUTXOSetBytes(ps.serializedUTXOSetStaging)
if err != nil { if err != nil {
return err return err
} }
err = dbTx.Put(pruningSerializedUTXOSetkey, utxoSetBytes) err = dbTx.Put(pruningSerializedUTXOSetKey, utxoSetBytes)
if err != nil { if err != nil {
return err return err
} }
@ -95,7 +148,7 @@ func (ps *pruningStore) PruningPointSerializedUTXOSet(dbContext model.DBReader)
return ps.serializedUTXOSetStaging, nil return ps.serializedUTXOSetStaging, nil
} }
dbPruningPointUTXOSetBytes, err := dbContext.Get(pruningSerializedUTXOSetkey) dbPruningPointUTXOSetBytes, err := dbContext.Get(pruningSerializedUTXOSetKey)
if err != nil { if err != nil {
return nil, err return nil, err
} }
@ -107,8 +160,8 @@ func (ps *pruningStore) PruningPointSerializedUTXOSet(dbContext model.DBReader)
return pruningPointUTXOSet, nil return pruningPointUTXOSet, nil
} }
func (ps *pruningStore) serializePruningPoint(pruningPoint *externalapi.DomainHash) ([]byte, error) { func (ps *pruningStore) serializeHash(hash *externalapi.DomainHash) ([]byte, error) {
return proto.Marshal(serialization.DomainHashToDbHash(pruningPoint)) return proto.Marshal(serialization.DomainHashToDbHash(hash))
} }
func (ps *pruningStore) deserializePruningPoint(pruningPointBytes []byte) (*externalapi.DomainHash, error) { func (ps *pruningStore) deserializePruningPoint(pruningPointBytes []byte) (*externalapi.DomainHash, error) {

View File

@ -105,6 +105,7 @@ func (f *factory) NewConsensus(dagParams *dagconfig.Params, db infrastructuredat
dbManager, dbManager,
dagTopologyManager, dagTopologyManager,
ghostdagDataStore, ghostdagDataStore,
reachabilityDataStore,
ghostdagManager) ghostdagManager)
pastMedianTimeManager := pastmediantimemanager.New( pastMedianTimeManager := pastmediantimemanager.New(
dagParams.TimestampDeviationTolerance, dagParams.TimestampDeviationTolerance,

View File

@ -14,6 +14,7 @@ type Consensus interface {
GetHashesBetween(lowHash, highHash *DomainHash) ([]*DomainHash, error) GetHashesBetween(lowHash, highHash *DomainHash) ([]*DomainHash, error)
GetMissingBlockBodyHashes(highHash *DomainHash) ([]*DomainHash, error) GetMissingBlockBodyHashes(highHash *DomainHash) ([]*DomainHash, error)
GetPruningPointUTXOSet(expectedPruningPointHash *DomainHash) ([]byte, error) GetPruningPointUTXOSet(expectedPruningPointHash *DomainHash) ([]byte, error)
PruningPoint() (*DomainHash, error)
ValidateAndInsertPruningPoint(newPruningPoint *DomainBlock, serializedUTXOSet []byte) error ValidateAndInsertPruningPoint(newPruningPoint *DomainBlock, serializedUTXOSet []byte) error
GetVirtualSelectedParent() (*DomainBlock, error) GetVirtualSelectedParent() (*DomainBlock, error)
CreateBlockLocator(lowHash, highHash *DomainHash, limit uint32) (BlockLocator, error) CreateBlockLocator(lowHash, highHash *DomainHash, limit uint32) (BlockLocator, error)
@ -21,5 +22,6 @@ type Consensus interface {
GetSyncInfo() (*SyncInfo, error) GetSyncInfo() (*SyncInfo, error)
Tips() ([]*DomainHash, error) Tips() ([]*DomainHash, error)
GetVirtualInfo() (*VirtualInfo, error) GetVirtualInfo() (*VirtualInfo, error)
IsValidPruningPoint(block *DomainHash) (bool, error)
GetVirtualSelectedParentChainFromBlock(blockHash *DomainHash) (*SelectedParentChainChanges, error) GetVirtualSelectedParentChainFromBlock(blockHash *DomainHash) (*SelectedParentChainChanges, error)
} }

View File

@ -2,8 +2,6 @@ package externalapi
// SyncInfo holds info about the current sync state of the consensus // SyncInfo holds info about the current sync state of the consensus
type SyncInfo struct { type SyncInfo struct {
IsAwaitingUTXOSet bool HeaderCount uint64
IBDRootUTXOBlockHash *DomainHash BlockCount uint64
HeaderCount uint64
BlockCount uint64
} }

View File

@ -5,8 +5,11 @@ import "github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
// PruningStore represents a store for the current pruning state // PruningStore represents a store for the current pruning state
type PruningStore interface { type PruningStore interface {
Store Store
Stage(pruningPointBlockHash *externalapi.DomainHash, pruningPointUTXOSetBytes []byte) StagePruningPoint(pruningPointBlockHash *externalapi.DomainHash, pruningPointUTXOSetBytes []byte)
StagePruningPointCandidate(candidate *externalapi.DomainHash)
IsStaged() bool IsStaged() bool
PruningPointCandidate(dbContext DBReader) (*externalapi.DomainHash, error)
HasPruningPointCandidate(dbContext DBReader) (bool, error)
PruningPoint(dbContext DBReader) (*externalapi.DomainHash, error) PruningPoint(dbContext DBReader) (*externalapi.DomainHash, error)
HasPruningPoint(dbContext DBReader) (bool, error) HasPruningPoint(dbContext DBReader) (bool, error)
PruningPointSerializedUTXOSet(dbContext DBReader) ([]byte, error) PruningPointSerializedUTXOSet(dbContext DBReader) ([]byte, error)

View File

@ -5,5 +5,5 @@ import "github.com/kaspanet/kaspad/domain/consensus/model/externalapi"
// PruningManager resolves and manages the current pruning point // PruningManager resolves and manages the current pruning point
type PruningManager interface { type PruningManager interface {
UpdatePruningPointByVirtual() error UpdatePruningPointByVirtual() error
CalculatePruningPointByHeaderSelectedTip() (*externalapi.DomainHash, error) IsValidPruningPoint(block *externalapi.DomainHash) (bool, error)
} }

View File

@ -10,16 +10,15 @@ import (
func (bp *blockProcessor) validateAndInsertPruningPoint(newPruningPoint *externalapi.DomainBlock, serializedUTXOSet []byte) error { func (bp *blockProcessor) validateAndInsertPruningPoint(newPruningPoint *externalapi.DomainBlock, serializedUTXOSet []byte) error {
log.Info("Checking that the given pruning point is the expected pruning point") log.Info("Checking that the given pruning point is the expected pruning point")
expectedNewPruningPointHash, err := bp.pruningManager.CalculatePruningPointByHeaderSelectedTip() newPruningPointHash := consensushashing.BlockHash(newPruningPoint)
isValidPruningPoint, err := bp.pruningManager.IsValidPruningPoint(newPruningPointHash)
if err != nil { if err != nil {
return err return err
} }
newPruningPointHash := consensushashing.BlockHash(newPruningPoint) if !isValidPruningPoint {
return errors.Wrapf(ruleerrors.ErrUnexpectedPruningPoint, "%s is not a valid pruning point",
if *expectedNewPruningPointHash != *newPruningPointHash { newPruningPointHash)
return errors.Wrapf(ruleerrors.ErrUnexpectedPruningPoint, "expected pruning point %s but got %s",
expectedNewPruningPointHash, newPruningPointHash)
} }
// We have to validate the pruning point block before we set the new pruning point in consensusStateManager. // We have to validate the pruning point block before we set the new pruning point in consensusStateManager.

View File

@ -41,7 +41,8 @@ func (csm *consensusStateManager) updatePruningPoint(newPruningPoint *externalap
if isViolatingFinality { if isViolatingFinality {
log.Warnf("Finality Violation Detected! The suggest pruning point %s violates finality!", newPruningPointHash) log.Warnf("Finality Violation Detected! The suggest pruning point %s violates finality!", newPruningPointHash)
return nil return errors.Wrapf(ruleerrors.ErrSuggestedPruningViolatesFinality, "%s cannot be a pruning point because "+
"it violates finality", newPruningPointHash)
} }
protoUTXOSet := &utxoserialization.ProtoUTXOSet{} protoUTXOSet := &utxoserialization.ProtoUTXOSet{}
@ -102,7 +103,7 @@ func (csm *consensusStateManager) updatePruningPoint(newPruningPoint *externalap
} }
log.Debugf("Staging the new pruning point and its UTXO set") log.Debugf("Staging the new pruning point and its UTXO set")
csm.pruningStore.Stage(newPruningPointHash, serializedUTXOSet) csm.pruningStore.StagePruningPoint(newPruningPointHash, serializedUTXOSet)
// Before we manually mark the new pruning point as valid, we validate that all of its transactions are valid // Before we manually mark the new pruning point as valid, we validate that all of its transactions are valid
// against the provided UTXO set. // against the provided UTXO set.

View File

@ -12,9 +12,10 @@ import (
type dagTraversalManager struct { type dagTraversalManager struct {
databaseContext model.DBReader databaseContext model.DBReader
dagTopologyManager model.DAGTopologyManager dagTopologyManager model.DAGTopologyManager
ghostdagDataStore model.GHOSTDAGDataStore ghostdagDataStore model.GHOSTDAGDataStore
ghostdagManager model.GHOSTDAGManager reachabilityDataStore model.ReachabilityDataStore
ghostdagManager model.GHOSTDAGManager
} }
// selectedParentIterator implements the `model.BlockIterator` API // selectedParentIterator implements the `model.BlockIterator` API
@ -45,12 +46,14 @@ func New(
databaseContext model.DBReader, databaseContext model.DBReader,
dagTopologyManager model.DAGTopologyManager, dagTopologyManager model.DAGTopologyManager,
ghostdagDataStore model.GHOSTDAGDataStore, ghostdagDataStore model.GHOSTDAGDataStore,
reachabilityDataStore model.ReachabilityDataStore,
ghostdagManager model.GHOSTDAGManager) model.DAGTraversalManager { ghostdagManager model.GHOSTDAGManager) model.DAGTraversalManager {
return &dagTraversalManager{ return &dagTraversalManager{
databaseContext: databaseContext, databaseContext: databaseContext,
dagTopologyManager: dagTopologyManager, dagTopologyManager: dagTopologyManager,
ghostdagDataStore: ghostdagDataStore, ghostdagDataStore: ghostdagDataStore,
ghostdagManager: ghostdagManager, reachabilityDataStore: reachabilityDataStore,
ghostdagManager: ghostdagManager,
} }
} }

View File

@ -9,21 +9,19 @@ import (
type selectedChildIterator struct { type selectedChildIterator struct {
databaseContext model.DBReader databaseContext model.DBReader
dagTopologyManager model.DAGTopologyManager dagTopologyManager model.DAGTopologyManager
highHash *externalapi.DomainHash
current *externalapi.DomainHash reachabilityDataStore model.ReachabilityDataStore
highHash *externalapi.DomainHash
current *externalapi.DomainHash
} }
func (s *selectedChildIterator) Next() bool { func (s *selectedChildIterator) Next() bool {
children, err := s.dagTopologyManager.Children(s.current) data, err := s.reachabilityDataStore.ReachabilityData(s.databaseContext, s.current)
if err != nil { if err != nil {
panic(err) panic(err)
} }
for _, child := range children { for _, child := range data.TreeNode.Children {
if *child == *model.VirtualBlockHash {
continue
}
isChildInSelectedParentChainOfHighHash, err := s.dagTopologyManager.IsInSelectedParentChainOf(child, s.highHash) isChildInSelectedParentChainOfHighHash, err := s.dagTopologyManager.IsInSelectedParentChainOf(child, s.highHash)
if err != nil { if err != nil {
panic(err) panic(err)
@ -51,9 +49,10 @@ func (dtm *dagTraversalManager) SelectedChildIterator(highHash, lowHash *externa
return nil, errors.Errorf("%s is not in the selected parent chain of %s", highHash, lowHash) return nil, errors.Errorf("%s is not in the selected parent chain of %s", highHash, lowHash)
} }
return &selectedChildIterator{ return &selectedChildIterator{
databaseContext: dtm.databaseContext, databaseContext: dtm.databaseContext,
dagTopologyManager: dtm.dagTopologyManager, dagTopologyManager: dtm.dagTopologyManager,
highHash: highHash, reachabilityDataStore: dtm.reachabilityDataStore,
current: lowHash, highHash: highHash,
current: lowHash,
}, nil }, nil
} }

View File

@ -26,10 +26,10 @@ type testJSON struct {
func TestPruning(t *testing.T) { func TestPruning(t *testing.T) {
expectedPruningPointByNet := map[string]map[string]string{ expectedPruningPointByNet := map[string]map[string]string{
"chain-for-test-pruning.json": { "chain-for-test-pruning.json": {
"kaspa-mainnet": "84", "kaspa-mainnet": "1582",
"kaspa-simnet": "84", "kaspa-simnet": "1582",
"kaspa-devnet": "84", "kaspa-devnet": "1582",
"kaspa-testnet": "84", "kaspa-testnet": "1582",
}, },
"dag-for-test-pruning.json": { "dag-for-test-pruning.json": {
"kaspa-mainnet": "503", "kaspa-mainnet": "503",
@ -100,9 +100,29 @@ func TestPruning(t *testing.T) {
blockIDToHash[dagBlock.ID] = blockHash blockIDToHash[dagBlock.ID] = blockHash
blockHashToID[*blockHash] = dagBlock.ID blockHashToID[*blockHash] = dagBlock.ID
pruningPoint, err := tc.PruningPoint()
if err != nil {
return err
}
pruningPointCandidate, err := tc.PruningStore().PruningPointCandidate(tc.DatabaseContext())
if err != nil {
return err
}
isValidPruningPoint, err := tc.IsValidPruningPoint(pruningPointCandidate)
if err != nil {
return err
}
shouldBeValid := *pruningPoint == *pruningPointCandidate
if isValidPruningPoint != shouldBeValid {
t.Fatalf("isValidPruningPoint is %t while expected %t", isValidPruningPoint, shouldBeValid)
}
} }
pruningPoint, err := tc.PruningStore().PruningPoint(tc.DatabaseContext()) pruningPoint, err := tc.PruningPoint()
if err != nil { if err != nil {
t.Fatalf("PruningPoint: %+v", err) t.Fatalf("PruningPoint: %+v", err)
} }

View File

@ -88,7 +88,12 @@ func (pm *pruningManager) UpdatePruningPointByVirtual() error {
} }
} }
currentP, err := pm.pruningStore.PruningPoint(pm.databaseContext) currentCandidate, err := pm.pruningPointCandidate()
if err != nil {
return err
}
currentCandidateGHOSTDAGData, err := pm.ghostdagDataStore.Get(pm.databaseContext, currentCandidate)
if err != nil { if err != nil {
return err return err
} }
@ -103,28 +108,71 @@ func (pm *pruningManager) UpdatePruningPointByVirtual() error {
return err return err
} }
currentPGhost, err := pm.ghostdagDataStore.Get(pm.databaseContext, currentP) currentPruningPoint, err := pm.pruningStore.PruningPoint(pm.databaseContext)
if err != nil {
return err
}
currentPBlueScore := currentPGhost.BlueScore()
// Because the pruning point changes only once per finality, then there's no need to even check for that if a finality interval hasn't passed.
if virtualSelectedParent.BlueScore() <= currentPBlueScore+pm.finalityInterval {
return nil
}
// This means the pruning point is still genesis.
if virtualSelectedParent.BlueScore() <= pm.pruningDepth+pm.finalityInterval {
return nil
}
// get Virtual(pruningDepth)
newPruningPoint, err := pm.calculatePruningPointFromBlock(model.VirtualBlockHash)
if err != nil { if err != nil {
return err return err
} }
if *newPruningPoint != *currentP { currentPruningPointGHOSTDAGData, err := pm.ghostdagDataStore.Get(pm.databaseContext, currentPruningPoint)
if err != nil {
return err
}
iterator, err := pm.dagTraversalManager.SelectedChildIterator(virtual.SelectedParent(), currentCandidate)
if err != nil {
return err
}
// Finding the next pruning point candidate: look for the latest
// selected child of the current candidate that is in depth of at
// least pm.pruningDepth blocks from the virtual selected parent.
//
// Note: Sometimes the current candidate is less than pm.pruningDepth
// from the virtual. This can happen only if the virtual blue score
// got smaller, because virtual blue score is not guaranteed to always
// increase (because sometimes a block with higher blue work can have
// lower blue score).
// In such cases we still keep the same candidate because it's guaranteed
// that a block that was once in depth of pm.pruningDepth cannot be
// reorged without causing a finality conflict first.
newCandidate := currentCandidate
newCandidateGHOSTDAGData := currentCandidateGHOSTDAGData
newPruningPoint := currentPruningPoint
newPruningPointGHOSTDAGData := currentPruningPointGHOSTDAGData
for iterator.Next() {
selectedChild := iterator.Get()
selectedChildGHOSTDAGData, err := pm.ghostdagDataStore.Get(pm.databaseContext, selectedChild)
if err != nil {
return err
}
if virtualSelectedParent.BlueScore()-selectedChildGHOSTDAGData.BlueScore() < pm.pruningDepth {
break
}
newCandidate = selectedChild
newCandidateGHOSTDAGData = selectedChildGHOSTDAGData
// We move the pruning point every time the candidate's finality score is
// bigger than the current pruning point finality score.
if pm.finalityScore(newCandidateGHOSTDAGData.BlueScore()) > pm.finalityScore(newPruningPointGHOSTDAGData.BlueScore()) {
newPruningPoint = newCandidate
newPruningPointGHOSTDAGData = newCandidateGHOSTDAGData
}
}
if *newCandidate != *currentCandidate {
pm.pruningStore.StagePruningPointCandidate(newCandidate)
}
// We move the pruning point every time the candidate's finality score is
// bigger than the current pruning point finality score.
if pm.finalityScore(newCandidateGHOSTDAGData.BlueScore()) <= pm.finalityScore(currentPruningPointGHOSTDAGData.BlueScore()) {
return nil
}
if *newPruningPoint != *currentPruningPoint {
err = pm.savePruningPoint(newPruningPoint) err = pm.savePruningPoint(newPruningPoint)
if err != nil { if err != nil {
return err return err
@ -214,7 +262,7 @@ func (pm *pruningManager) savePruningPoint(blockHash *externalapi.DomainHash) er
if err != nil { if err != nil {
return err return err
} }
pm.pruningStore.Stage(blockHash, serializedUtxo) pm.pruningStore.StagePruningPoint(blockHash, serializedUtxo)
return nil return nil
} }
@ -237,28 +285,68 @@ func (pm *pruningManager) deleteBlock(blockHash *externalapi.DomainHash) (alread
return false, nil return false, nil
} }
func (pm *pruningManager) CalculatePruningPointByHeaderSelectedTip() (*externalapi.DomainHash, error) { func (pm *pruningManager) IsValidPruningPoint(block *externalapi.DomainHash) (bool, error) {
if *pm.genesisHash == *block {
return true, nil
}
headersSelectedTip, err := pm.headerSelectedTipStore.HeadersSelectedTip(pm.databaseContext) headersSelectedTip, err := pm.headerSelectedTipStore.HeadersSelectedTip(pm.databaseContext)
if err != nil { if err != nil {
return nil, err return false, err
} }
return pm.calculatePruningPointFromBlock(headersSelectedTip) // A pruning point has to be in the selected chain of the headers selected tip.
headersSelectedTipGHOSTDAGData, err := pm.ghostdagDataStore.Get(pm.databaseContext, headersSelectedTip)
if err != nil {
return false, err
}
isInSelectedParentChainOfHeadersSelectedTip, err := pm.dagTopologyManager.IsInSelectedParentChainOf(block,
headersSelectedTip)
if err != nil {
return false, err
}
if !isInSelectedParentChainOfHeadersSelectedTip {
return false, nil
}
ghostdagData, err := pm.ghostdagDataStore.Get(pm.databaseContext, block)
if err != nil {
return false, err
}
// A pruning point has to be at depth of at least pm.pruningDepth
if headersSelectedTipGHOSTDAGData.BlueScore()-ghostdagData.BlueScore() < pm.pruningDepth {
return false, nil
}
selectedParentGHOSTDAGData, err := pm.ghostdagDataStore.Get(pm.databaseContext, ghostdagData.SelectedParent())
if err != nil {
return false, err
}
// A pruning point has to be the lowest chain block with a certain finality score, so
// if the block selected parent has the same finality score it means it cannot be a
// pruning point.
if pm.finalityScore(ghostdagData.BlueScore()) == pm.finalityScore(selectedParentGHOSTDAGData.BlueScore()) {
return false, nil
}
return true, nil
} }
func (pm *pruningManager) calculatePruningPointFromBlock(blockHash *externalapi.DomainHash) (*externalapi.DomainHash, error) { func (pm *pruningManager) pruningPointCandidate() (*externalapi.DomainHash, error) {
ghostdagData, err := pm.ghostdagDataStore.Get(pm.databaseContext, blockHash) hasPruningPointCandidate, err := pm.pruningStore.HasPruningPointCandidate(pm.databaseContext)
if err != nil { if err != nil {
return nil, err return nil, err
} }
targetBlueScore := uint64(0) if !hasPruningPointCandidate {
if ghostdagData.BlueScore() > pm.pruningDepth { return pm.genesisHash, nil
// The target blue is calculated by calculating ghostdagData.BlueScore() - pm.pruningDepth and rounding
// down with the precision of finality interval.
targetBlueScore = ((ghostdagData.BlueScore() - pm.pruningDepth) / pm.finalityInterval) * pm.finalityInterval
} }
return pm.dagTraversalManager.LowestChainBlockAboveOrEqualToBlueScore(blockHash, targetBlueScore)
return pm.pruningStore.PruningPointCandidate(pm.databaseContext)
} }
func serializeUTXOSetIterator(iter model.ReadOnlyUTXOSetIterator) ([]byte, error) { func serializeUTXOSetIterator(iter model.ReadOnlyUTXOSetIterator) ([]byte, error) {
@ -268,3 +356,9 @@ func serializeUTXOSetIterator(iter model.ReadOnlyUTXOSetIterator) ([]byte, error
} }
return proto.Marshal(serializedUtxo) return proto.Marshal(serializedUtxo)
} }
// finalityScore is the number of finality intervals passed since
// the given block.
func (pm *pruningManager) finalityScore(blueScore uint64) uint64 {
return blueScore / pm.finalityInterval
}

View File

@ -5,44 +5,15 @@ import (
) )
func (sm *syncManager) syncInfo() (*externalapi.SyncInfo, error) { func (sm *syncManager) syncInfo() (*externalapi.SyncInfo, error) {
isAwaitingUTXOSet, ibdRootUTXOBlockHash, err := sm.isAwaitingUTXOSet()
if err != nil {
return nil, err
}
headerCount := sm.getHeaderCount() headerCount := sm.getHeaderCount()
blockCount := sm.getBlockCount() blockCount := sm.getBlockCount()
return &externalapi.SyncInfo{ return &externalapi.SyncInfo{
IsAwaitingUTXOSet: isAwaitingUTXOSet, HeaderCount: headerCount,
IBDRootUTXOBlockHash: ibdRootUTXOBlockHash, BlockCount: blockCount,
HeaderCount: headerCount,
BlockCount: blockCount,
}, nil }, nil
} }
func (sm *syncManager) isAwaitingUTXOSet() (isAwaitingUTXOSet bool, ibdRootUTXOBlockHash *externalapi.DomainHash,
err error) {
pruningPointByHeaders, err := sm.pruningManager.CalculatePruningPointByHeaderSelectedTip()
if err != nil {
return false, nil, err
}
pruningPoint, err := sm.pruningStore.PruningPoint(sm.databaseContext)
if err != nil {
return false, nil, err
}
// If the pruning point by headers is different from the current point
// it means we need to request the new pruning point UTXO set.
if *pruningPoint != *pruningPointByHeaders {
return true, pruningPointByHeaders, nil
}
return false, nil, nil
}
func (sm *syncManager) getHeaderCount() uint64 { func (sm *syncManager) getHeaderCount() uint64 {
return sm.blockHeaderStore.Count() return sm.blockHeaderStore.Count()
} }

View File

@ -242,6 +242,8 @@ var (
ErrBlockIsTooMuchInTheFuture = newRuleError("ErrBlockIsTooMuchInTheFuture") ErrBlockIsTooMuchInTheFuture = newRuleError("ErrBlockIsTooMuchInTheFuture")
ErrUnexpectedPruningPoint = newRuleError("ErrUnexpectedPruningPoint") ErrUnexpectedPruningPoint = newRuleError("ErrUnexpectedPruningPoint")
ErrSuggestedPruningViolatesFinality = newRuleError("ErrSuggestedPruningViolatesFinality")
) )
// RuleError identifies a rule violation. It is used to indicate that // RuleError identifies a rule violation. It is used to indicate that

View File

@ -30,6 +30,8 @@ message KaspadMessage {
IBDRootUTXOSetAndBlockMessage ibdRootUTXOSetAndBlock = 25; IBDRootUTXOSetAndBlockMessage ibdRootUTXOSetAndBlock = 25;
RequestIBDBlocksMessage requestIBDBlocks = 26; RequestIBDBlocksMessage requestIBDBlocks = 26;
IBDRootNotFoundMessage ibdRootNotFound = 27; IBDRootNotFoundMessage ibdRootNotFound = 27;
RequestIBDRootHash requestIBDRootHash = 28;
IBDRootHash ibdRootHash = 29;
GetCurrentNetworkRequestMessage getCurrentNetworkRequest = 1001; GetCurrentNetworkRequestMessage getCurrentNetworkRequest = 1001;
GetCurrentNetworkResponseMessage getCurrentNetworkResponse = 1002; GetCurrentNetworkResponseMessage getCurrentNetworkResponse = 1002;
@ -297,6 +299,17 @@ message IBDRootNotFoundMessage{
} }
// IBDRootNotFoundMessage end // IBDRootNotFoundMessage end
// RequestIBDRootHash start
message RequestIBDRootHash{
}
// RequestIBDRootHash end
// IBDRootHash start
message IBDRootHash{
Hash hash = 1;
}
// IBDRootHash end
service P2P { service P2P {
rpc MessageStream (stream KaspadMessage) returns (stream KaspadMessage) {} rpc MessageStream (stream KaspadMessage) returns (stream KaspadMessage) {}
} }

View File

@ -0,0 +1,19 @@
package protowire
import "github.com/kaspanet/kaspad/app/appmessage"
func (x *KaspadMessage_IbdRootHash) toAppMessage() (appmessage.Message, error) {
hash, err := x.IbdRootHash.Hash.toDomain()
if err != nil {
return nil, err
}
return &appmessage.MsgIBDRootHash{Hash: hash}, nil
}
func (x *KaspadMessage_IbdRootHash) fromAppMessage(msgIBDRootHash *appmessage.MsgIBDRootHash) error {
x.IbdRootHash = &IBDRootHash{
Hash: domainHashToProto(msgIBDRootHash.Hash),
}
return nil
}

View File

@ -0,0 +1,11 @@
package protowire
import "github.com/kaspanet/kaspad/app/appmessage"
func (x *KaspadMessage_RequestIBDRootHash) toAppMessage() (appmessage.Message, error) {
return &appmessage.MsgRequestIBDRootHash{}, nil
}
func (x *KaspadMessage_RequestIBDRootHash) fromAppMessage(_ *appmessage.MsgRequestIBDRootHash) error {
return nil
}

View File

@ -232,6 +232,22 @@ func toP2PPayload(message appmessage.Message) (isKaspadMessage_Payload, error) {
} }
return payload, nil return payload, nil
case *appmessage.MsgRequestIBDRootHash:
payload := new(KaspadMessage_RequestIBDRootHash)
err := payload.fromAppMessage(message)
if err != nil {
return nil, err
}
return payload, nil
case *appmessage.MsgIBDRootHash:
payload := new(KaspadMessage_IbdRootHash)
err := payload.fromAppMessage(message)
if err != nil {
return nil, err
}
return payload, nil
default: default:
return nil, nil return nil, nil
} }