Skip to content
Merged
12 changes: 12 additions & 0 deletions plugin/evm/extension/config.go
Original file line number Diff line number Diff line change
Expand Up @@ -10,6 +10,7 @@ import (

"github.com/ava-labs/subnet-evm/plugin/evm/message"
"github.com/ava-labs/subnet-evm/plugin/evm/sync"
"github.com/ava-labs/subnet-evm/sync/handlers"
)

var (
Expand All @@ -19,6 +20,17 @@ var (
errNilClock = errors.New("nil clock")
)

// LeafRequestConfig is the configuration to handle leaf requests
// in the network and syncer
type LeafRequestConfig struct {
// LeafType is the type of the leaf node
LeafType message.NodeType
// MetricName is the name of the metric to use for the leaf request
MetricName string
// Handler is the handler to use for the leaf request
Handler handlers.LeafRequestHandler
}

// Config is the configuration for the VM extension
type Config struct {
// SyncSummaryProvider is the sync summary provider to use
Expand Down
5 changes: 2 additions & 3 deletions plugin/evm/message/handler.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,9 +15,8 @@ var _ RequestHandler = NoopRequestHandler{}
// Must have methods in format of handleType(context.Context, ids.NodeID, uint32, request Type) error
// so that the Request object of relevant Type can invoke its respective handle method
// on this struct.
// Also see GossipHandler for implementation style.
type RequestHandler interface {
HandleStateTrieLeafsRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, leafsRequest LeafsRequest) ([]byte, error)
HandleLeafsRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, leafsRequest LeafsRequest) ([]byte, error)
HandleBlockRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, request BlockRequest) ([]byte, error)
HandleCodeRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, codeRequest CodeRequest) ([]byte, error)
}
Expand All @@ -33,7 +32,7 @@ type ResponseHandler interface {

type NoopRequestHandler struct{}

func (NoopRequestHandler) HandleStateTrieLeafsRequest(context.Context, ids.NodeID, uint32, LeafsRequest) ([]byte, error) {
func (NoopRequestHandler) HandleLeafsRequest(context.Context, ids.NodeID, uint32, LeafsRequest) ([]byte, error) {
return nil, nil
}

Expand Down
29 changes: 20 additions & 9 deletions plugin/evm/message/leafs_request.go
Original file line number Diff line number Diff line change
Expand Up @@ -13,27 +13,38 @@ import (

const MaxCodeHashesPerRequest = 5

var _ Request = LeafsRequest{}
// NodeType outlines the trie that a leaf node belongs to
// handlers.LeafsRequestHandler uses this information to determine
// which trie type to fetch the information from
type NodeType uint8

const (
StateTrieNode = NodeType(1)
StateTrieKeyLength = common.HashLength
)

// LeafsRequest is a request to receive trie leaves at specified Root within Start and End byte range
// Limit outlines maximum number of leaves to returns starting at Start
// NodeType outlines which trie to read from state/atomic.
// NOTE: NodeType is not serialized to avoid breaking changes. We rely on the single used node type (StateTrieNode).
type LeafsRequest struct {
Root common.Hash `serialize:"true"`
Account common.Hash `serialize:"true"`
Start []byte `serialize:"true"`
End []byte `serialize:"true"`
Limit uint16 `serialize:"true"`
Root common.Hash `serialize:"true"`
Account common.Hash `serialize:"true"`
Start []byte `serialize:"true"`
End []byte `serialize:"true"`
Limit uint16 `serialize:"true"`
NodeType NodeType `serialize:"false"`
}

func (l LeafsRequest) String() string {
return fmt.Sprintf(
"LeafsRequest(Root=%s, Account=%s, Start=%s, End %s, Limit=%d)",
l.Root, l.Account, common.Bytes2Hex(l.Start), common.Bytes2Hex(l.End), l.Limit,
"LeafsRequest(Root=%s, Account=%s, Start=%s, End=%s, Limit=%d, NodeType=%d)",
l.Root, l.Account, common.Bytes2Hex(l.Start), common.Bytes2Hex(l.End), l.Limit, l.NodeType,
)
}

func (l LeafsRequest) Handle(ctx context.Context, nodeID ids.NodeID, requestID uint32, handler RequestHandler) ([]byte, error) {
return handler.HandleStateTrieLeafsRequest(ctx, nodeID, requestID, l)
return handler.HandleLeafsRequest(ctx, nodeID, requestID, l)
}

// LeafsResponse is a response to a LeafsRequest
Expand Down
62 changes: 0 additions & 62 deletions plugin/evm/message/leafs_request_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -4,13 +4,10 @@
package message

import (
"bytes"
"context"
"encoding/base64"
"math/rand"
"testing"

"github.com/ava-labs/avalanchego/ids"
"github.com/ava-labs/libevm/common"
"github.com/stretchr/testify/assert"
)
Expand Down Expand Up @@ -105,62 +102,3 @@ func TestMarshalLeafsResponse(t *testing.T) {
assert.False(t, l.More) // make sure it is not serialized
assert.Equal(t, leafsResponse.ProofVals, l.ProofVals)
}

func TestLeafsRequestValidation(t *testing.T) {
mockRequestHandler := &mockHandler{}

tests := map[string]struct {
request LeafsRequest
assertResponse func(t *testing.T)
}{
"node type StateTrieNode": {
request: LeafsRequest{
Root: common.BytesToHash([]byte("some hash goes here")),
Start: bytes.Repeat([]byte{0x00}, common.HashLength),
End: bytes.Repeat([]byte{0xff}, common.HashLength),
Limit: 10,
},
assertResponse: func(t *testing.T) {
assert.True(t, mockRequestHandler.handleStateTrieCalled)
assert.False(t, mockRequestHandler.handleBlockRequestCalled)
assert.False(t, mockRequestHandler.handleCodeRequestCalled)
},
},
}
for name, test := range tests {
t.Run(name, func(t *testing.T) {
_, _ = test.request.Handle(context.Background(), ids.GenerateTestNodeID(), 1, mockRequestHandler)
test.assertResponse(t)
mockRequestHandler.reset()
})
}
}

var _ RequestHandler = (*mockHandler)(nil)

type mockHandler struct {
handleStateTrieCalled,
handleBlockRequestCalled,
handleCodeRequestCalled bool
}

func (m *mockHandler) HandleStateTrieLeafsRequest(context.Context, ids.NodeID, uint32, LeafsRequest) ([]byte, error) {
m.handleStateTrieCalled = true
return nil, nil
}

func (m *mockHandler) HandleBlockRequest(context.Context, ids.NodeID, uint32, BlockRequest) ([]byte, error) {
m.handleBlockRequestCalled = true
return nil, nil
}

func (m *mockHandler) HandleCodeRequest(context.Context, ids.NodeID, uint32, CodeRequest) ([]byte, error) {
m.handleCodeRequestCalled = true
return nil, nil
}

func (m *mockHandler) reset() {
m.handleStateTrieCalled = false
m.handleBlockRequestCalled = false
m.handleCodeRequestCalled = false
}
38 changes: 30 additions & 8 deletions plugin/evm/network_handler.go
Original file line number Diff line number Diff line change
Expand Up @@ -9,7 +9,7 @@ import (
"github.com/ava-labs/avalanchego/codec"
"github.com/ava-labs/avalanchego/ids"
"github.com/ava-labs/libevm/ethdb"
"github.com/ava-labs/libevm/metrics"
"github.com/ava-labs/libevm/log"
"github.com/ava-labs/libevm/triedb"

"github.com/ava-labs/subnet-evm/plugin/evm/message"
Expand All @@ -20,29 +20,51 @@ import (

var _ message.RequestHandler = (*networkHandler)(nil)

type LeafHandlers map[message.NodeType]syncHandlers.LeafRequestHandler

type networkHandler struct {
leafRequestHandler *syncHandlers.LeafsRequestHandler
leafRequestHandlers LeafHandlers
blockRequestHandler *syncHandlers.BlockRequestHandler
codeRequestHandler *syncHandlers.CodeRequestHandler
}

type LeafRequestTypeConfig struct {
NodeType message.NodeType
NodeKeyLen int
TrieDB *triedb.Database
UseSnapshots bool
MetricName string
}

// newNetworkHandler constructs the handler for serving network requests.
func newNetworkHandler(
provider syncHandlers.SyncDataProvider,
diskDB ethdb.KeyValueReader,
evmTrieDB *triedb.Database,
networkCodec codec.Manager,
) message.RequestHandler {
syncStats := syncStats.NewHandlerStats(metrics.Enabled)
leafRequestHandlers LeafHandlers,
syncStats syncStats.HandlerStats,
) *networkHandler {
return &networkHandler{
leafRequestHandler: syncHandlers.NewLeafsRequestHandler(evmTrieDB, nil, networkCodec, syncStats),
leafRequestHandlers: leafRequestHandlers,
blockRequestHandler: syncHandlers.NewBlockRequestHandler(provider, networkCodec, syncStats),
codeRequestHandler: syncHandlers.NewCodeRequestHandler(diskDB, networkCodec, syncStats),
}
}

func (n networkHandler) HandleStateTrieLeafsRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, leafsRequest message.LeafsRequest) ([]byte, error) {
return n.leafRequestHandler.OnLeafsRequest(ctx, nodeID, requestID, leafsRequest)
func (n networkHandler) HandleLeafsRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, leafsRequest message.LeafsRequest) ([]byte, error) {
nodeType := leafsRequest.NodeType
// TODO(JonathanOppenheimer):Handle legacy requests where NodeType was not serialized (defaults to 0)
// In this interim period, we treat NodeType 0 as StateTrieNode
if nodeType == 0 {
nodeType = message.StateTrieNode
}

handler, ok := n.leafRequestHandlers[nodeType]
if !ok {
log.Debug("node type is not recognised, dropping request", "nodeID", nodeID, "requestID", requestID, "nodeType", leafsRequest.NodeType)
return nil, nil
}
return handler.OnLeafsRequest(ctx, nodeID, requestID, leafsRequest)
}

func (n networkHandler) HandleBlockRequest(ctx context.Context, nodeID ids.NodeID, requestID uint32, blockRequest message.BlockRequest) ([]byte, error) {
Expand Down
77 changes: 50 additions & 27 deletions plugin/evm/vm.go
Original file line number Diff line number Diff line change
Expand Up @@ -75,6 +75,7 @@ import (
"github.com/ava-labs/subnet-evm/precompile/precompileconfig"
"github.com/ava-labs/subnet-evm/rpc"
"github.com/ava-labs/subnet-evm/sync/client/stats"
"github.com/ava-labs/subnet-evm/sync/handlers"
"github.com/ava-labs/subnet-evm/triedb/hashdb"
"github.com/ava-labs/subnet-evm/warp"

Expand All @@ -87,6 +88,7 @@ import (
subnetevmlog "github.com/ava-labs/subnet-evm/plugin/evm/log"
vmsync "github.com/ava-labs/subnet-evm/plugin/evm/sync"
statesyncclient "github.com/ava-labs/subnet-evm/sync/client"
handlerstats "github.com/ava-labs/subnet-evm/sync/handlers/stats"
avalancheRPC "github.com/gorilla/rpc/v2"
)

Expand Down Expand Up @@ -504,11 +506,9 @@ func (vm *VM) Initialize(
warpHandler := acp118.NewCachedHandler(meteredCache, vm.warpBackend, vm.ctx.WarpSigner)
vm.Network.AddHandler(p2p.SignatureRequestHandlerID, warpHandler)

vm.setAppRequestHandlers()

vm.stateSyncDone = make(chan struct{})

return vm.initializeStateSyncClient(lastAcceptedHeight)
return vm.initializeStateSync(lastAcceptedHeight)
}

func parseGenesis(ctx *snow.Context, genesisBytes []byte, upgradeBytes []byte, airdropFile string) (*core.Genesis, error) {
Expand Down Expand Up @@ -652,7 +652,47 @@ func (vm *VM) initializeChain(lastAcceptedHash common.Hash, ethConfig ethconfig.
// initializeStateSyncClient initializes the client for performing state sync.
// If state sync is disabled, this function will wipe any ongoing summary from
// disk to ensure that we do not continue syncing from an invalid snapshot.
func (vm *VM) initializeStateSyncClient(lastAcceptedHeight uint64) error {
func (vm *VM) initializeStateSync(lastAcceptedHeight uint64) error {
// Create standalone EVM TrieDB (read only) for serving leafs requests.
// We create a standalone TrieDB here, so that it has a standalone cache from the one
// used by the node when processing blocks.
evmTrieDB := triedb.NewDatabase(
vm.chaindb,
&triedb.Config{
DBOverride: hashdb.Config{
CleanCacheSize: vm.config.StateSyncServerTrieCache * units.MiB,
}.BackendConstructor,
},
)

// register default leaf request handler for state trie
syncStats := handlerstats.GetOrRegisterHandlerStats(metrics.Enabled)
stateLeafRequestConfig := &extension.LeafRequestConfig{
LeafType: message.StateTrieNode,
MetricName: "sync_state_trie_leaves",
Handler: handlers.NewLeafsRequestHandler(evmTrieDB,
message.StateTrieKeyLength,
vm.blockChain, vm.networkCodec,
syncStats,
),
}

leafHandlers := make(LeafHandlers)
leafHandlers[stateLeafRequestConfig.LeafType] = stateLeafRequestConfig.Handler
// TODO(JonathanOppenheimer): Register handler for NodeType 0 (legacy/unspecified) for
// backward compatibility while NodeType is not serialized
leafHandlers[0] = stateLeafRequestConfig.Handler

networkHandler := newNetworkHandler(
vm.blockChain,
vm.chaindb,
vm.networkCodec,
leafHandlers,
syncStats,
)
vm.Network.SetRequestHandler(networkHandler)

vm.Server = vmsync.NewServer(vm.blockChain, vm.extensionConfig.SyncSummaryProvider, vm.config.StateSyncCommitInterval) // parse nodeIDs from state sync IDs in vm config
// parse nodeIDs from state sync IDs in vm config
var stateSyncIDs []ids.NodeID
if vm.config.StateSyncEnabled && len(vm.config.StateSyncIDs) > 0 {
Expand All @@ -667,15 +707,19 @@ func (vm *VM) initializeStateSyncClient(lastAcceptedHeight uint64) error {
}
}

// Initialize the state sync client
leafMetricsNames := make(map[message.NodeType]string)
leafMetricsNames[stateLeafRequestConfig.LeafType] = stateLeafRequestConfig.MetricName

vm.Client = vmsync.NewClient(&vmsync.ClientConfig{
StateSyncDone: vm.stateSyncDone,
Chain: vm.eth,
State: vm.State,
StateSyncDone: vm.stateSyncDone,
Client: statesyncclient.NewClient(
&statesyncclient.ClientConfig{
NetworkClient: vm.Network,
Codec: vm.networkCodec,
Stats: stats.NewClientSyncerStats(),
Stats: stats.NewClientSyncerStats(leafMetricsNames),
StateSyncNodeIDs: stateSyncIDs,
BlockParser: vm,
},
Expand Down Expand Up @@ -906,27 +950,6 @@ func (vm *VM) onNormalOperationsStarted() error {
return nil
}

// setAppRequestHandlers sets the request handlers for the VM to serve state sync
// requests.
func (vm *VM) setAppRequestHandlers() {
// Create standalone EVM TrieDB (read only) for serving leafs requests.
// We create a standalone TrieDB here, so that it has a standalone cache from the one
// used by the node when processing blocks.
evmTrieDB := triedb.NewDatabase(
vm.chaindb,
&triedb.Config{
DBOverride: hashdb.Config{
CleanCacheSize: vm.config.StateSyncServerTrieCache * units.MiB,
}.BackendConstructor,
},
)

networkHandler := newNetworkHandler(vm.blockChain, vm.chaindb, evmTrieDB, vm.networkCodec)
vm.Network.SetRequestHandler(networkHandler)

vm.Server = vmsync.NewServer(vm.blockChain, vm.extensionConfig.SyncSummaryProvider, vm.config.StateSyncCommitInterval)
}

func (vm *VM) WaitForEvent(ctx context.Context) (commonEng.Message, error) {
vm.builderLock.Lock()
builder := vm.builder
Expand Down
4 changes: 3 additions & 1 deletion sync/README.md
Original file line number Diff line number Diff line change
Expand Up @@ -60,6 +60,8 @@ The following steps are executed by the VM to sync its state from peers (see `st
1. Update in-memory and on-disk pointers.

Steps 3 and 4 involve syncing tries. To sync trie data, the VM will send a series of `LeafRequests` to its peers. Each request specifies:
- Type of trie (`NodeType`):
- `statesync.StateTrieNode` (account trie and storage tries share the same database)
- `Root` of the trie to sync,
- `Start` and `End` specify a range of keys.

Expand Down Expand Up @@ -112,4 +114,4 @@ While state sync is faster than normal bootstrapping, the process may take sever
| `state-sync-skip-resume` | `bool` | set to true to avoid resuming an ongoing sync | `false` |
| `state-sync-min-blocks` | `uint64` | Minimum number of blocks the chain must be ahead of local state to prefer state sync over bootstrapping | `300,000` |
| `state-sync-server-trie-cache` | `int` | Size of trie cache to serve state sync data in MB. Should be set to multiples of `64`. | `64` |
| `state-sync-ids` | `string` | a comma separated list of `NodeID-` prefixed node IDs to sync data from. If not provided, peers are randomly selected. | |
| `state-sync-ids` | `string` | a comma separated list of `NodeID-` prefixed node IDs to sync data from. If not provided, peers are randomly selected. | |
Loading
Loading