2020-08-19 05:12:58 +00:00
|
|
|
// Copyright © 2020 Vulcanize, Inc
|
|
|
|
//
|
|
|
|
// This program is free software: you can redistribute it and/or modify
|
|
|
|
// it under the terms of the GNU Affero General Public License as published by
|
|
|
|
// the Free Software Foundation, either version 3 of the License, or
|
|
|
|
// (at your option) any later version.
|
|
|
|
//
|
|
|
|
// This program is distributed in the hope that it will be useful,
|
|
|
|
// but WITHOUT ANY WARRANTY; without even the implied warranty of
|
|
|
|
// MERCHANTABILITY or FITNESS FOR A PARTICULAR PURPOSE. See the
|
|
|
|
// GNU Affero General Public License for more details.
|
|
|
|
//
|
|
|
|
// You should have received a copy of the GNU Affero General Public License
|
|
|
|
// along with this program. If not, see <http://www.gnu.org/licenses/>.
|
|
|
|
|
|
|
|
package statediff
|
|
|
|
|
|
|
|
import (
|
|
|
|
"bytes"
|
|
|
|
"fmt"
|
|
|
|
"math/big"
|
2020-08-19 05:57:57 +00:00
|
|
|
"sync"
|
2021-10-21 18:00:30 +00:00
|
|
|
"time"
|
2020-08-19 05:12:58 +00:00
|
|
|
|
2023-09-29 20:27:08 +00:00
|
|
|
"github.com/cerc-io/plugeth-statediff"
|
|
|
|
"github.com/cerc-io/plugeth-statediff/adapt"
|
|
|
|
"github.com/cerc-io/plugeth-statediff/indexer/interfaces"
|
|
|
|
sdtypes "github.com/cerc-io/plugeth-statediff/types"
|
2020-08-19 05:12:58 +00:00
|
|
|
"github.com/ethereum/go-ethereum/common"
|
|
|
|
"github.com/ethereum/go-ethereum/core/types"
|
|
|
|
"github.com/ethereum/go-ethereum/p2p"
|
|
|
|
"github.com/ethereum/go-ethereum/rlp"
|
|
|
|
"github.com/ethereum/go-ethereum/rpc"
|
2020-08-19 16:35:09 +00:00
|
|
|
"github.com/sirupsen/logrus"
|
2020-11-10 09:07:59 +00:00
|
|
|
|
2022-09-20 18:00:10 +00:00
|
|
|
"github.com/cerc-io/eth-statediff-service/pkg/prom"
|
2020-08-19 05:12:58 +00:00
|
|
|
)
|
|
|
|
|
2021-10-25 19:06:05 +00:00
|
|
|
const defaultQueueSize = 1024
|
2020-08-19 05:12:58 +00:00
|
|
|
|
|
|
|
// Service is the underlying struct for the state diffing service
|
|
|
|
type Service struct {
|
|
|
|
// Used to build the state diff objects
|
2023-09-29 20:27:08 +00:00
|
|
|
builder statediff.Builder
|
2022-05-17 09:05:11 +00:00
|
|
|
// Used to read data from LevelDB
|
2021-10-25 19:06:05 +00:00
|
|
|
lvlDBReader Reader
|
2020-08-19 05:12:58 +00:00
|
|
|
// Used to signal shutdown of the service
|
2021-10-25 19:26:41 +00:00
|
|
|
quitChan chan struct{}
|
2020-11-10 09:07:59 +00:00
|
|
|
// Interface for publishing statediffs as PG-IPLD objects
|
2021-11-13 00:45:08 +00:00
|
|
|
indexer interfaces.StateDiffIndexer
|
2021-10-25 19:06:05 +00:00
|
|
|
// range queue
|
|
|
|
queue chan RangeRequest
|
|
|
|
// number of ranges we can work over concurrently
|
|
|
|
workers uint
|
|
|
|
// ranges configured locally
|
|
|
|
preruns []RangeRequest
|
2020-08-19 05:12:58 +00:00
|
|
|
}
|
|
|
|
|
2020-09-04 16:50:49 +00:00
|
|
|
// NewStateDiffService creates a new Service
|
2023-09-29 20:27:08 +00:00
|
|
|
func NewStateDiffService(lvlDBReader Reader, indexer interfaces.StateDiffIndexer, conf ServiceConfig) *Service {
|
|
|
|
builder := statediff.NewBuilder(adapt.GethStateView(lvlDBReader.StateDB()))
|
|
|
|
builder.SetSubtrieWorkers(conf.TrieWorkers)
|
2021-10-25 19:06:05 +00:00
|
|
|
if conf.WorkerQueueSize == 0 {
|
|
|
|
conf.WorkerQueueSize = defaultQueueSize
|
|
|
|
}
|
2020-08-19 05:12:58 +00:00
|
|
|
return &Service{
|
2020-08-19 05:57:57 +00:00
|
|
|
lvlDBReader: lvlDBReader,
|
2023-09-29 20:27:08 +00:00
|
|
|
builder: builder,
|
2020-11-10 09:07:59 +00:00
|
|
|
indexer: indexer,
|
2021-10-25 19:06:05 +00:00
|
|
|
workers: conf.ServiceWorkers,
|
|
|
|
queue: make(chan RangeRequest, conf.WorkerQueueSize),
|
|
|
|
preruns: conf.PreRuns,
|
2023-09-29 20:27:08 +00:00
|
|
|
}
|
2020-08-19 05:12:58 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// Protocols exports the services p2p protocols, this service has none
|
|
|
|
func (sds *Service) Protocols() []p2p.Protocol {
|
|
|
|
return []p2p.Protocol{}
|
|
|
|
}
|
|
|
|
|
2020-09-04 16:50:49 +00:00
|
|
|
// APIs returns the RPC descriptors the Service offers
|
2020-08-19 05:12:58 +00:00
|
|
|
func (sds *Service) APIs() []rpc.API {
|
|
|
|
return []rpc.API{
|
|
|
|
{
|
2020-09-04 16:50:49 +00:00
|
|
|
Namespace: APIName,
|
|
|
|
Version: APIVersion,
|
2020-08-19 05:12:58 +00:00
|
|
|
Service: NewPublicStateDiffAPI(sds),
|
|
|
|
Public: true,
|
|
|
|
},
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2023-09-29 20:27:08 +00:00
|
|
|
func segmentRange(workers, start, stop uint64, params statediff.Params) []RangeRequest {
|
2022-10-04 19:16:44 +00:00
|
|
|
segmentSize := ((stop - start) + 1) / workers
|
|
|
|
remainder := ((stop - start) + 1) % workers
|
|
|
|
numOfSegments := workers
|
|
|
|
if remainder > 0 {
|
|
|
|
numOfSegments++
|
|
|
|
}
|
|
|
|
segments := make([]RangeRequest, numOfSegments)
|
|
|
|
for i := range segments {
|
|
|
|
end := start + segmentSize - 1
|
|
|
|
if end > stop {
|
|
|
|
end = stop
|
|
|
|
}
|
|
|
|
segments[i] = RangeRequest{start, end, params}
|
|
|
|
start = end + 1
|
|
|
|
}
|
|
|
|
return segments
|
|
|
|
}
|
|
|
|
|
2021-10-27 15:40:08 +00:00
|
|
|
// Run does a one-off processing run on the provided RangeRequests + any pre-runs, exiting afterwards
|
2022-08-31 04:31:54 +00:00
|
|
|
func (sds *Service) Run(rngs []RangeRequest, parallel bool) error {
|
2021-10-27 15:40:08 +00:00
|
|
|
for _, preRun := range sds.preruns {
|
2022-10-04 19:16:44 +00:00
|
|
|
// if the rangeSize is smaller than the number of workers
|
|
|
|
// make sure we do synchronous processing to avoid quantization issues
|
|
|
|
rangeSize := (preRun.Stop - preRun.Start) + 1
|
|
|
|
numWorkers := uint64(sds.workers)
|
|
|
|
if rangeSize < numWorkers {
|
|
|
|
parallel = false
|
|
|
|
}
|
2022-08-31 04:31:54 +00:00
|
|
|
if parallel {
|
|
|
|
logrus.Infof("parallel processing prerun range (%d, %d) (%d blocks) divided into %d sized chunks with %d workers", preRun.Start, preRun.Stop,
|
2022-10-04 19:16:44 +00:00
|
|
|
rangeSize, rangeSize/numWorkers, numWorkers)
|
|
|
|
workChan := make(chan RangeRequest)
|
|
|
|
quitChan := make(chan struct{})
|
|
|
|
// spin up numWorkers number of worker goroutines
|
2022-08-31 04:31:54 +00:00
|
|
|
wg := new(sync.WaitGroup)
|
2022-10-04 19:16:44 +00:00
|
|
|
for i := 0; i < int(numWorkers); i++ {
|
2022-08-31 04:31:54 +00:00
|
|
|
wg.Add(1)
|
|
|
|
go func(id int) {
|
|
|
|
defer wg.Done()
|
2022-10-04 19:16:44 +00:00
|
|
|
for {
|
|
|
|
select {
|
|
|
|
case workerSegment := <-workChan:
|
|
|
|
for j := workerSegment.Start; j <= workerSegment.Stop; j++ {
|
|
|
|
if err := sds.WriteStateDiffAt(j, workerSegment.Params); err != nil {
|
|
|
|
logrus.Errorf("error writing statediff at height %d in range (%d, %d) : %v", id, workerSegment.Start, workerSegment.Stop, err)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
logrus.Infof("prerun worker %d finished processing range (%d, %d)", id, workerSegment.Start, workerSegment.Stop)
|
|
|
|
case <-quitChan:
|
|
|
|
return
|
2022-08-31 04:31:54 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}(i)
|
|
|
|
}
|
2022-10-04 19:16:44 +00:00
|
|
|
// break range up into segments
|
|
|
|
segments := segmentRange(numWorkers, preRun.Start, preRun.Stop, preRun.Params)
|
|
|
|
// send the segments to the work channel
|
|
|
|
for _, segment := range segments {
|
|
|
|
workChan <- segment
|
|
|
|
}
|
|
|
|
close(quitChan)
|
2022-08-31 04:31:54 +00:00
|
|
|
wg.Wait()
|
|
|
|
} else {
|
|
|
|
logrus.Infof("sequential processing prerun range (%d, %d)", preRun.Start, preRun.Stop)
|
|
|
|
for i := preRun.Start; i <= preRun.Stop; i++ {
|
|
|
|
if err := sds.WriteStateDiffAt(i, preRun.Params); err != nil {
|
|
|
|
return fmt.Errorf("error writing statediff at height %d in range (%d, %d) : %v", i, preRun.Start, preRun.Stop, err)
|
|
|
|
}
|
2021-10-27 15:40:08 +00:00
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
sds.preruns = nil
|
2022-09-21 19:33:15 +00:00
|
|
|
// At present this code is never called so we have not written the parallel version:
|
2021-10-27 15:40:08 +00:00
|
|
|
for _, rng := range rngs {
|
2022-08-31 04:31:54 +00:00
|
|
|
logrus.Infof("processing requested range (%d, %d)", rng.Start, rng.Stop)
|
2021-10-27 15:40:08 +00:00
|
|
|
for i := rng.Start; i <= rng.Stop; i++ {
|
|
|
|
if err := sds.WriteStateDiffAt(i, rng.Params); err != nil {
|
|
|
|
return fmt.Errorf("error writing statediff at height %d in range (%d, %d) : %v", i, rng.Start, rng.Stop, err)
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}
|
|
|
|
return nil
|
|
|
|
}
|
|
|
|
|
2020-08-19 05:12:58 +00:00
|
|
|
// Loop is an empty service loop for awaiting rpc requests
|
2021-10-25 19:26:41 +00:00
|
|
|
func (sds *Service) Loop(wg *sync.WaitGroup) error {
|
|
|
|
if sds.quitChan != nil {
|
|
|
|
return fmt.Errorf("service loop is already running")
|
|
|
|
}
|
2021-10-27 15:40:08 +00:00
|
|
|
|
2021-10-25 19:26:41 +00:00
|
|
|
sds.quitChan = make(chan struct{})
|
2021-10-25 19:06:05 +00:00
|
|
|
for i := 0; i < int(sds.workers); i++ {
|
|
|
|
wg.Add(1)
|
|
|
|
go func(id int) {
|
|
|
|
defer wg.Done()
|
|
|
|
for {
|
|
|
|
select {
|
|
|
|
case blockRange := <-sds.queue:
|
2023-09-29 20:27:08 +00:00
|
|
|
log := logrus.WithField("range", blockRange).WithField("worker", id)
|
|
|
|
log.Debug("processing range")
|
2021-10-25 19:26:41 +00:00
|
|
|
prom.DecQueuedRanges()
|
2021-10-25 19:54:55 +00:00
|
|
|
for j := blockRange.Start; j <= blockRange.Stop; j++ {
|
2021-10-25 19:06:05 +00:00
|
|
|
if err := sds.WriteStateDiffAt(j, blockRange.Params); err != nil {
|
2023-09-29 20:27:08 +00:00
|
|
|
log.Errorf("error writing statediff at block %d: %v", j, err)
|
2021-10-25 19:06:05 +00:00
|
|
|
}
|
|
|
|
select {
|
|
|
|
case <-sds.quitChan:
|
2023-09-29 20:27:08 +00:00
|
|
|
log.Infof("closing service worker (last processed block: %d)", j)
|
2021-10-25 19:06:05 +00:00
|
|
|
return
|
|
|
|
default:
|
2023-09-29 20:27:08 +00:00
|
|
|
log.Infof("Finished processing block %d", j)
|
2021-10-25 19:06:05 +00:00
|
|
|
}
|
|
|
|
}
|
2023-09-29 20:27:08 +00:00
|
|
|
log.Debugf("Finished processing range")
|
2021-10-25 19:06:05 +00:00
|
|
|
case <-sds.quitChan:
|
2023-09-29 20:27:08 +00:00
|
|
|
logrus.Debugf("closing the statediff service loop worker %d", id)
|
2021-10-25 19:06:05 +00:00
|
|
|
return
|
|
|
|
}
|
|
|
|
}
|
|
|
|
}(i)
|
|
|
|
}
|
|
|
|
for _, preRun := range sds.preruns {
|
2021-10-25 19:26:41 +00:00
|
|
|
if err := sds.WriteStateDiffsInRange(preRun.Start, preRun.Stop, preRun.Params); err != nil {
|
|
|
|
close(sds.quitChan)
|
|
|
|
return err
|
|
|
|
}
|
2020-08-19 05:12:58 +00:00
|
|
|
}
|
2021-10-25 19:26:41 +00:00
|
|
|
return nil
|
2020-08-19 05:12:58 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// StateDiffAt returns a state diff object payload at the specific blockheight
|
|
|
|
// This operation cannot be performed back past the point of db pruning; it requires an archival node for historical data
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) StateDiffAt(blockNumber uint64, params statediff.Params) (*statediff.Payload, error) {
|
2020-08-19 05:12:58 +00:00
|
|
|
currentBlock, err := sds.lvlDBReader.GetBlockByNumber(blockNumber)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2021-11-09 01:08:36 +00:00
|
|
|
logrus.Infof("sending state diff at block %d", blockNumber)
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2022-07-12 11:08:05 +00:00
|
|
|
// compute leaf paths of watched addresses in the params
|
2022-07-01 07:59:48 +00:00
|
|
|
params.ComputeWatchedAddressesLeafPaths()
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2020-08-19 05:12:58 +00:00
|
|
|
if blockNumber == 0 {
|
|
|
|
return sds.processStateDiff(currentBlock, common.Hash{}, params)
|
|
|
|
}
|
|
|
|
parentBlock, err := sds.lvlDBReader.GetBlockByHash(currentBlock.ParentHash())
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
return sds.processStateDiff(currentBlock, parentBlock.Root(), params)
|
|
|
|
}
|
|
|
|
|
2021-02-27 18:39:16 +00:00
|
|
|
// StateDiffFor returns a state diff object payload for the specific blockhash
|
|
|
|
// This operation cannot be performed back past the point of db pruning; it requires an archival node for historical data
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) StateDiffFor(blockHash common.Hash, params statediff.Params) (*statediff.Payload, error) {
|
2021-02-27 18:39:16 +00:00
|
|
|
currentBlock, err := sds.lvlDBReader.GetBlockByHash(blockHash)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2023-09-29 20:27:08 +00:00
|
|
|
logrus.Infof("sending state diff at block %s", blockHash)
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2022-07-12 11:08:05 +00:00
|
|
|
// compute leaf paths of watched addresses in the params
|
2022-07-01 07:59:48 +00:00
|
|
|
params.ComputeWatchedAddressesLeafPaths()
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2021-02-27 18:39:16 +00:00
|
|
|
if currentBlock.NumberU64() == 0 {
|
|
|
|
return sds.processStateDiff(currentBlock, common.Hash{}, params)
|
|
|
|
}
|
|
|
|
parentBlock, err := sds.lvlDBReader.GetBlockByHash(currentBlock.ParentHash())
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
return sds.processStateDiff(currentBlock, parentBlock.Root(), params)
|
|
|
|
}
|
|
|
|
|
2020-08-19 05:12:58 +00:00
|
|
|
// processStateDiff method builds the state diff payload from the current block, parent state root, and provided params
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) processStateDiff(currentBlock *types.Block, parentRoot common.Hash, params statediff.Params) (*statediff.Payload, error) {
|
|
|
|
stateDiff, err := sds.builder.BuildStateDiffObject(statediff.Args{
|
2020-08-19 05:12:58 +00:00
|
|
|
BlockHash: currentBlock.Hash(),
|
|
|
|
BlockNumber: currentBlock.Number(),
|
2020-09-04 16:50:49 +00:00
|
|
|
OldStateRoot: parentRoot,
|
|
|
|
NewStateRoot: currentBlock.Root(),
|
2020-08-19 05:12:58 +00:00
|
|
|
}, params)
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2022-05-11 09:51:23 +00:00
|
|
|
stateDiffRlp, err := rlp.EncodeToBytes(&stateDiff)
|
2020-08-19 05:12:58 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
2020-08-19 19:34:05 +00:00
|
|
|
logrus.Infof("state diff object at block %d is %d bytes in length", currentBlock.Number().Uint64(), len(stateDiffRlp))
|
2020-08-19 05:12:58 +00:00
|
|
|
return sds.newPayload(stateDiffRlp, currentBlock, params)
|
|
|
|
}
|
|
|
|
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) newPayload(stateObject []byte, block *types.Block, params statediff.Params) (*statediff.Payload, error) {
|
|
|
|
payload := &statediff.Payload{
|
2020-08-19 05:12:58 +00:00
|
|
|
StateObjectRlp: stateObject,
|
|
|
|
}
|
|
|
|
if params.IncludeBlock {
|
|
|
|
blockBuff := new(bytes.Buffer)
|
|
|
|
if err := block.EncodeRLP(blockBuff); err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
payload.BlockRlp = blockBuff.Bytes()
|
|
|
|
}
|
|
|
|
if params.IncludeTD {
|
|
|
|
var err error
|
2020-08-19 05:57:57 +00:00
|
|
|
payload.TotalDifficulty, err = sds.lvlDBReader.GetTdByHash(block.Hash())
|
2020-08-19 05:12:58 +00:00
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
}
|
|
|
|
if params.IncludeReceipts {
|
|
|
|
receiptBuff := new(bytes.Buffer)
|
|
|
|
receipts, err := sds.lvlDBReader.GetReceiptsByHash(block.Hash())
|
|
|
|
if err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
if err := rlp.Encode(receiptBuff, receipts); err != nil {
|
|
|
|
return nil, err
|
|
|
|
}
|
|
|
|
payload.ReceiptsRlp = receiptBuff.Bytes()
|
|
|
|
}
|
|
|
|
return payload, nil
|
|
|
|
}
|
|
|
|
|
|
|
|
// Start is used to begin the service
|
2020-11-16 09:49:59 +00:00
|
|
|
func (sds *Service) Start() error {
|
2020-08-19 16:35:09 +00:00
|
|
|
logrus.Info("starting statediff service")
|
2021-10-25 19:26:41 +00:00
|
|
|
return sds.Loop(new(sync.WaitGroup))
|
2020-08-19 05:12:58 +00:00
|
|
|
}
|
|
|
|
|
|
|
|
// Stop is used to close down the service
|
|
|
|
func (sds *Service) Stop() error {
|
2020-08-19 16:35:09 +00:00
|
|
|
logrus.Info("stopping statediff service")
|
2021-10-25 19:06:05 +00:00
|
|
|
close(sds.quitChan)
|
2020-08-19 05:12:58 +00:00
|
|
|
return nil
|
|
|
|
}
|
2020-11-10 09:07:59 +00:00
|
|
|
|
|
|
|
// WriteStateDiffAt writes a state diff at the specific blockheight directly to the database
|
|
|
|
// This operation cannot be performed back past the point of db pruning; it requires an archival node
|
|
|
|
// for historical data
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) WriteStateDiffAt(blockNumber uint64, params statediff.Params) error {
|
2021-11-09 01:08:36 +00:00
|
|
|
logrus.Infof("Writing state diff at block %d", blockNumber)
|
2021-10-21 18:00:30 +00:00
|
|
|
t := time.Now()
|
2020-11-10 09:07:59 +00:00
|
|
|
currentBlock, err := sds.lvlDBReader.GetBlockByNumber(blockNumber)
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2022-07-12 11:08:05 +00:00
|
|
|
// compute leaf paths of watched addresses in the params
|
2022-07-01 07:59:48 +00:00
|
|
|
params.ComputeWatchedAddressesLeafPaths()
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2020-11-10 09:07:59 +00:00
|
|
|
parentRoot := common.Hash{}
|
|
|
|
if blockNumber != 0 {
|
|
|
|
parentBlock, err := sds.lvlDBReader.GetBlockByHash(currentBlock.ParentHash())
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
parentRoot = parentBlock.Root()
|
|
|
|
}
|
2021-10-21 18:00:30 +00:00
|
|
|
return sds.writeStateDiff(currentBlock, parentRoot, params, t)
|
2020-11-10 09:07:59 +00:00
|
|
|
}
|
|
|
|
|
2021-02-27 18:39:16 +00:00
|
|
|
// WriteStateDiffFor writes a state diff for the specific blockHash directly to the database
|
|
|
|
// This operation cannot be performed back past the point of db pruning; it requires an archival node
|
|
|
|
// for historical data
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) WriteStateDiffFor(blockHash common.Hash, params statediff.Params) error {
|
|
|
|
logrus.Infof("Writing state diff for block %s", blockHash)
|
2021-10-21 18:00:30 +00:00
|
|
|
t := time.Now()
|
2021-02-27 18:39:16 +00:00
|
|
|
currentBlock, err := sds.lvlDBReader.GetBlockByHash(blockHash)
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2022-07-12 11:08:05 +00:00
|
|
|
// compute leaf paths of watched addresses in the params
|
2022-07-01 07:59:48 +00:00
|
|
|
params.ComputeWatchedAddressesLeafPaths()
|
2022-05-12 10:21:46 +00:00
|
|
|
|
2021-02-27 18:39:16 +00:00
|
|
|
parentRoot := common.Hash{}
|
|
|
|
if currentBlock.NumberU64() != 0 {
|
|
|
|
parentBlock, err := sds.lvlDBReader.GetBlockByHash(currentBlock.ParentHash())
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
parentRoot = parentBlock.Root()
|
|
|
|
}
|
2021-10-21 18:00:30 +00:00
|
|
|
return sds.writeStateDiff(currentBlock, parentRoot, params, t)
|
2021-02-27 18:39:16 +00:00
|
|
|
}
|
|
|
|
|
2020-11-10 09:07:59 +00:00
|
|
|
// Writes a state diff from the current block, parent state root, and provided params
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) writeStateDiff(block *types.Block, parentRoot common.Hash, params statediff.Params, t time.Time) error {
|
2020-11-10 09:07:59 +00:00
|
|
|
var totalDifficulty *big.Int
|
|
|
|
var receipts types.Receipts
|
|
|
|
var err error
|
|
|
|
if params.IncludeTD {
|
|
|
|
totalDifficulty, err = sds.lvlDBReader.GetTdByHash(block.Hash())
|
|
|
|
}
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
if params.IncludeReceipts {
|
|
|
|
receipts, err = sds.lvlDBReader.GetReceiptsByHash(block.Hash())
|
|
|
|
}
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
2021-10-21 18:00:30 +00:00
|
|
|
height := block.Number().Int64()
|
|
|
|
prom.SetLastLoadedHeight(height)
|
2021-10-24 23:36:49 +00:00
|
|
|
prom.SetTimeMetric(prom.T_BLOCK_LOAD, time.Now().Sub(t))
|
2021-10-21 18:00:30 +00:00
|
|
|
t = time.Now()
|
2020-11-10 09:07:59 +00:00
|
|
|
tx, err := sds.indexer.PushBlock(block, receipts, totalDifficulty)
|
|
|
|
if err != nil {
|
|
|
|
return err
|
|
|
|
}
|
|
|
|
// defer handling of commit/rollback for any return case
|
2023-10-04 07:44:43 +00:00
|
|
|
defer tx.RollbackOnFailure(err)
|
|
|
|
|
|
|
|
var nodeMtx, ipldMtx sync.Mutex
|
2023-04-09 16:01:05 +00:00
|
|
|
output := func(node sdtypes.StateLeafNode) error {
|
2023-10-04 07:44:43 +00:00
|
|
|
nodeMtx.Lock()
|
|
|
|
defer nodeMtx.Unlock()
|
2021-11-19 00:43:16 +00:00
|
|
|
return sds.indexer.PushStateNode(tx, node, block.Hash().String())
|
2020-11-10 09:07:59 +00:00
|
|
|
}
|
2023-10-04 07:44:43 +00:00
|
|
|
ipldOutput := func(c sdtypes.IPLD) error {
|
|
|
|
ipldMtx.Lock()
|
|
|
|
defer ipldMtx.Unlock()
|
2023-04-09 16:01:05 +00:00
|
|
|
return sds.indexer.PushIPLD(tx, c)
|
2020-11-10 09:07:59 +00:00
|
|
|
}
|
2021-10-24 23:36:49 +00:00
|
|
|
prom.SetTimeMetric(prom.T_BLOCK_PROCESSING, time.Now().Sub(t))
|
2021-10-21 18:00:30 +00:00
|
|
|
t = time.Now()
|
2023-09-29 20:27:08 +00:00
|
|
|
err = sds.builder.WriteStateDiff(statediff.Args{
|
2020-11-10 09:07:59 +00:00
|
|
|
NewStateRoot: block.Root(),
|
|
|
|
OldStateRoot: parentRoot,
|
2023-09-29 20:27:08 +00:00
|
|
|
BlockNumber: block.Number(),
|
|
|
|
BlockHash: block.Hash(),
|
2023-10-04 07:44:43 +00:00
|
|
|
}, params, output, ipldOutput)
|
2021-10-24 23:36:49 +00:00
|
|
|
prom.SetTimeMetric(prom.T_STATE_PROCESSING, time.Now().Sub(t))
|
2021-10-21 18:00:30 +00:00
|
|
|
t = time.Now()
|
2023-09-29 20:27:08 +00:00
|
|
|
err = tx.Submit()
|
2021-10-21 18:00:30 +00:00
|
|
|
prom.SetLastProcessedHeight(height)
|
2021-10-24 23:36:49 +00:00
|
|
|
prom.SetTimeMetric(prom.T_POSTGRES_TX_COMMIT, time.Now().Sub(t))
|
2021-10-21 18:00:30 +00:00
|
|
|
return err
|
2020-11-10 09:07:59 +00:00
|
|
|
}
|
2021-10-25 19:06:05 +00:00
|
|
|
|
|
|
|
// WriteStateDiffsInRange adds a RangeRequest to the work queue
|
2023-09-29 20:27:08 +00:00
|
|
|
func (sds *Service) WriteStateDiffsInRange(start, stop uint64, params statediff.Params) error {
|
2021-10-25 19:06:05 +00:00
|
|
|
if stop < start {
|
|
|
|
return fmt.Errorf("invalid block range (%d, %d): stop height must be greater or equal to start height", start, stop)
|
|
|
|
}
|
|
|
|
blocked := time.NewTimer(30 * time.Second)
|
|
|
|
select {
|
|
|
|
case sds.queue <- RangeRequest{Start: start, Stop: stop, Params: params}:
|
2021-10-25 19:26:41 +00:00
|
|
|
prom.IncQueuedRanges()
|
2023-09-29 20:27:08 +00:00
|
|
|
logrus.Infof("Added range (%d, %d) to the worker queue", start, stop)
|
2021-10-25 19:06:05 +00:00
|
|
|
return nil
|
|
|
|
case <-blocked.C:
|
|
|
|
return fmt.Errorf("unable to add range (%d, %d) to the worker queue", start, stop)
|
|
|
|
}
|
|
|
|
}
|