diff options
Diffstat (limited to 'plugin/evm')
-rw-r--r-- | plugin/evm/block.go | 9 | ||||
-rw-r--r-- | plugin/evm/database.go | 9 | ||||
-rw-r--r-- | plugin/evm/export_tx.go | 2 | ||||
-rw-r--r-- | plugin/evm/import_tx.go | 2 | ||||
-rw-r--r-- | plugin/evm/service.go | 23 | ||||
-rw-r--r-- | plugin/evm/tx.go | 2 | ||||
-rw-r--r-- | plugin/evm/user.go | 2 | ||||
-rw-r--r-- | plugin/evm/vm.go | 146 |
8 files changed, 113 insertions, 82 deletions
diff --git a/plugin/evm/block.go b/plugin/evm/block.go index 134fa29..ff1f6ae 100644 --- a/plugin/evm/block.go +++ b/plugin/evm/block.go @@ -8,7 +8,8 @@ import ( "fmt" "github.com/ava-labs/coreth/core/types" - "github.com/ava-labs/go-ethereum/rlp" + "github.com/ethereum/go-ethereum/log" + "github.com/ethereum/go-ethereum/rlp" "github.com/ava-labs/avalanchego/ids" "github.com/ava-labs/avalanchego/snow/choices" @@ -30,7 +31,7 @@ func (b *Block) ID() ids.ID { return b.id } func (b *Block) Accept() error { vm := b.vm - vm.ctx.Log.Verbo("Block %s is accepted", b.id) + log.Trace(fmt.Sprintf("Block %s is accepted", b.ID())) vm.updateStatus(b.id, choices.Accepted) if err := vm.acceptedDB.Put(b.ethBlock.Number().Bytes(), b.id.Bytes()); err != nil { return err @@ -50,7 +51,7 @@ func (b *Block) Accept() error { // Reject implements the snowman.Block interface func (b *Block) Reject() error { - b.vm.ctx.Log.Verbo("Block %s is rejected", b.ID()) + log.Trace(fmt.Sprintf("Block %s is rejected", b.ID())) b.vm.updateStatus(b.ID(), choices.Rejected) return nil } @@ -68,10 +69,8 @@ func (b *Block) Status() choices.Status { func (b *Block) Parent() snowman.Block { parentID := ids.NewID(b.ethBlock.ParentHash()) if block := b.vm.getBlock(parentID); block != nil { - b.vm.ctx.Log.Verbo("Parent(%s) has status: %s", parentID, block.Status()) return block } - b.vm.ctx.Log.Verbo("Parent(%s) has status: %s", parentID, choices.Unknown) return &missing.Block{BlkID: parentID} } diff --git a/plugin/evm/database.go b/plugin/evm/database.go index aedbc9b..18890fa 100644 --- a/plugin/evm/database.go +++ b/plugin/evm/database.go @@ -6,7 +6,7 @@ package evm import ( "errors" - "github.com/ava-labs/go-ethereum/ethdb" + "github.com/ethereum/go-ethereum/ethdb" "github.com/ava-labs/avalanchego/database" ) @@ -47,11 +47,8 @@ func (db Database) Sync() error { return errOpNotSupported } func (db Database) NewBatch() ethdb.Batch { return Batch{db.Database.NewBatch()} } // NewIterator implements ethdb.Database -func (db Database) NewIterator() ethdb.Iterator { return db.Database.NewIterator() } - -// NewIteratorWithPrefix implements ethdb.Database -func (db Database) NewIteratorWithPrefix(prefix []byte) ethdb.Iterator { - return db.NewIteratorWithPrefix(prefix) +func (db Database) NewIterator(prefix []byte, start []byte) ethdb.Iterator { + return db.NewIteratorWithStartAndPrefix(start, prefix) } // NewIteratorWithStart implements ethdb.Database diff --git a/plugin/evm/export_tx.go b/plugin/evm/export_tx.go index 43f858c..0487c44 100644 --- a/plugin/evm/export_tx.go +++ b/plugin/evm/export_tx.go @@ -8,7 +8,7 @@ import ( "math/big" "github.com/ava-labs/coreth/core/state" - "github.com/ava-labs/go-ethereum/log" + "github.com/ethereum/go-ethereum/log" "github.com/ava-labs/avalanchego/chains/atomic" "github.com/ava-labs/avalanchego/database" diff --git a/plugin/evm/import_tx.go b/plugin/evm/import_tx.go index c6ad2d9..7d17c4e 100644 --- a/plugin/evm/import_tx.go +++ b/plugin/evm/import_tx.go @@ -16,7 +16,7 @@ import ( "github.com/ava-labs/avalanchego/utils/math" "github.com/ava-labs/avalanchego/vms/components/avax" "github.com/ava-labs/avalanchego/vms/secp256k1fx" - "github.com/ava-labs/go-ethereum/common" + "github.com/ethereum/go-ethereum/common" ) // UnsignedImportTx is an unsigned ImportTx diff --git a/plugin/evm/service.go b/plugin/evm/service.go index 33429b5..a3325b7 100644 --- a/plugin/evm/service.go +++ b/plugin/evm/service.go @@ -14,15 +14,16 @@ import ( "github.com/ava-labs/coreth" - "github.com/ava-labs/coreth/core/types" "github.com/ava-labs/avalanchego/api" "github.com/ava-labs/avalanchego/utils/constants" "github.com/ava-labs/avalanchego/utils/crypto" "github.com/ava-labs/avalanchego/utils/formatting" "github.com/ava-labs/avalanchego/utils/json" - "github.com/ava-labs/go-ethereum/common" - "github.com/ava-labs/go-ethereum/common/hexutil" - ethcrypto "github.com/ava-labs/go-ethereum/crypto" + "github.com/ava-labs/coreth/core/types" + "github.com/ethereum/go-ethereum/common" + "github.com/ethereum/go-ethereum/common/hexutil" + ethcrypto "github.com/ethereum/go-ethereum/crypto" + "github.com/ethereum/go-ethereum/log" ) const ( @@ -87,7 +88,7 @@ func (api *SnowmanAPI) GetAcceptedFront(ctx context.Context) (*GetAcceptedFrontR // GetGenesisBalance returns the current funds in the genesis func (api *DebugAPI) GetGenesisBalance(ctx context.Context) (*hexutil.Big, error) { lastAccepted := api.vm.getLastAccepted() - api.vm.ctx.Log.Verbo("Currently accepted block front: %s", lastAccepted.ethBlock.Hash().Hex()) + log.Trace(fmt.Sprintf("Currently accepted block front: %s", lastAccepted.ethBlock.Hash().Hex())) state, err := api.vm.chain.BlockState(lastAccepted.ethBlock) if err != nil { return nil, err @@ -97,7 +98,7 @@ func (api *DebugAPI) GetGenesisBalance(ctx context.Context) (*hexutil.Big, error // SpendGenesis funds func (api *DebugAPI) SpendGenesis(ctx context.Context, nonce uint64) error { - api.vm.ctx.Log.Info("Spending the genesis") + log.Info("Spending the genesis") value := big.NewInt(1000000000000) gasLimit := 21000 @@ -127,7 +128,7 @@ func (api *DebugAPI) SpendGenesis(ctx context.Context, nonce uint64) error { // IssueBlock to the chain func (api *DebugAPI) IssueBlock(ctx context.Context) error { - api.vm.ctx.Log.Info("Issuing a new block") + log.Info("Issuing a new block") return api.vm.tryBlockGen() } @@ -146,7 +147,7 @@ type ExportKeyReply struct { // ExportKey returns a private key from the provided user func (service *AvaxAPI) ExportKey(r *http.Request, args *ExportKeyArgs, reply *ExportKeyReply) error { - service.vm.ctx.Log.Info("Platform: ExportKey called") + log.Info("EVM: ExportKey called") address, err := service.vm.ParseEthAddress(args.Address) if err != nil { @@ -176,7 +177,7 @@ type ImportKeyArgs struct { // ImportKey adds a private key to the provided user func (service *AvaxAPI) ImportKey(r *http.Request, args *ImportKeyArgs, reply *api.JsonAddress) error { - service.vm.ctx.Log.Info("Platform: ImportKey called for user '%s'", args.Username) + log.Info(fmt.Sprintf("EVM: ImportKey called for user '%s'", args.Username)) if !strings.HasPrefix(args.PrivateKey, constants.SecretKeyPrefix) { return fmt.Errorf("private key missing %s prefix", constants.SecretKeyPrefix) @@ -228,7 +229,7 @@ type ImportAVAXArgs struct { // ImportAVAX issues a transaction to import AVAX from the X-chain. The AVAX // must have already been exported from the X-Chain. func (service *AvaxAPI) ImportAVAX(_ *http.Request, args *ImportAVAXArgs, response *api.JsonTxID) error { - service.vm.ctx.Log.Info("Platform: ImportAVAX called") + log.Info("EVM: ImportAVAX called") chainID, err := service.vm.ctx.BCLookup.Lookup(args.SourceChain) if err != nil { @@ -277,7 +278,7 @@ type ExportAVAXArgs struct { // ExportAVAX exports AVAX from the P-Chain to the X-Chain // It must be imported on the X-Chain to complete the transfer func (service *AvaxAPI) ExportAVAX(_ *http.Request, args *ExportAVAXArgs, response *api.JsonTxID) error { - service.vm.ctx.Log.Info("Platform: ExportAVAX called") + log.Info("EVM: ExportAVAX called") if args.Amount == 0 { return errors.New("argument 'amount' must be > 0") diff --git a/plugin/evm/tx.go b/plugin/evm/tx.go index e573580..e39a053 100644 --- a/plugin/evm/tx.go +++ b/plugin/evm/tx.go @@ -17,7 +17,7 @@ import ( "github.com/ava-labs/avalanchego/utils/hashing" "github.com/ava-labs/avalanchego/vms/components/verify" "github.com/ava-labs/avalanchego/vms/secp256k1fx" - "github.com/ava-labs/go-ethereum/common" + "github.com/ethereum/go-ethereum/common" ) // Max size of memo field diff --git a/plugin/evm/user.go b/plugin/evm/user.go index cfa302b..b751634 100644 --- a/plugin/evm/user.go +++ b/plugin/evm/user.go @@ -10,7 +10,7 @@ import ( "github.com/ava-labs/avalanchego/database" "github.com/ava-labs/avalanchego/ids" "github.com/ava-labs/avalanchego/utils/crypto" - "github.com/ava-labs/go-ethereum/common" + "github.com/ethereum/go-ethereum/common" ) // Key in the database whose corresponding value is the list of diff --git a/plugin/evm/vm.go b/plugin/evm/vm.go index 9ab4781..200a08d 100644 --- a/plugin/evm/vm.go +++ b/plugin/evm/vm.go @@ -22,11 +22,13 @@ import ( "github.com/ava-labs/coreth/node" "github.com/ava-labs/coreth/params" - "github.com/ava-labs/go-ethereum/common" - "github.com/ava-labs/go-ethereum/rlp" - "github.com/ava-labs/go-ethereum/rpc" + "github.com/ethereum/go-ethereum/common" + "github.com/ethereum/go-ethereum/event" + "github.com/ethereum/go-ethereum/log" + "github.com/ethereum/go-ethereum/rlp" + "github.com/ethereum/go-ethereum/rpc" - ethcrypto "github.com/ava-labs/go-ethereum/crypto" + ethcrypto "github.com/ethereum/go-ethereum/crypto" avalancheRPC "github.com/gorilla/rpc/v2" @@ -152,20 +154,21 @@ func init() { type VM struct { ctx *snow.Context - chainID *big.Int - networkID uint64 - genesisHash common.Hash - chain *coreth.ETHChain - chaindb Database - newBlockChan chan *Block - networkChan chan<- commonEng.Message - newTxPoolHeadChan chan core.NewTxPoolHeadEvent + chainID *big.Int + networkID uint64 + genesisHash common.Hash + chain *coreth.ETHChain + chaindb Database + newBlockChan chan *Block + networkChan chan<- commonEng.Message + newMinedBlockSub *event.TypeMuxSubscription acceptedDB database.Database - txPoolStabilizedHead common.Hash - txPoolStabilizedOk chan struct{} - txPoolStabilizedLock sync.Mutex + txPoolStabilizedHead common.Hash + txPoolStabilizedOk chan struct{} + txPoolStabilizedLock sync.Mutex + txPoolStabilizedShutdownChan chan struct{} metalock sync.Mutex blockCache, blockStatusCache cache.LRU @@ -181,12 +184,15 @@ type VM struct { genlock sync.Mutex txSubmitChan <-chan struct{} atomicTxSubmitChan chan struct{} + shutdownSubmitChan chan struct{} codec codec.Codec clock timer.Clock txFee uint64 pendingAtomicTxs chan *Tx blockAtomicInputCache cache.LRU + shutdownWg sync.WaitGroup + fx secp256k1fx.Fx } @@ -243,6 +249,10 @@ func (vm *VM) Initialize( config := eth.DefaultConfig config.ManualCanonical = true config.Genesis = g + // disable the experimental snapshot feature from geth + config.TrieCleanCache += config.SnapshotCache + config.SnapshotCache = 0 + config.Miner.ManualMining = true config.Miner.DisableUncle = true @@ -287,7 +297,7 @@ func (vm *VM) Initialize( return nil, nil }) chain.SetOnSealFinish(func(block *types.Block) error { - vm.ctx.Log.Verbo("EVM sealed a block") + log.Trace("EVM sealed a block") blk := &Block{ id: ids.NewID(block.Hash()), @@ -295,6 +305,7 @@ func (vm *VM) Initialize( vm: vm, } if blk.Verify() != nil { + vm.newBlockChan <- nil return errInvalidBlock } vm.newBlockChan <- blk @@ -338,26 +349,15 @@ func (vm *VM) Initialize( vm.bdTimerState = bdTimerStateLong vm.bdGenWaitFlag = true - vm.newTxPoolHeadChan = make(chan core.NewTxPoolHeadEvent, 1) vm.txPoolStabilizedOk = make(chan struct{}, 1) + vm.txPoolStabilizedShutdownChan = make(chan struct{}, 1) // Signal goroutine to shutdown // TODO: read size from options vm.pendingAtomicTxs = make(chan *Tx, 1024) vm.atomicTxSubmitChan = make(chan struct{}, 1) - chain.GetTxPool().SubscribeNewHeadEvent(vm.newTxPoolHeadChan) - // TODO: shutdown this go routine - go ctx.Log.RecoverAndPanic(func() { - for { - select { - case h := <-vm.newTxPoolHeadChan: - vm.txPoolStabilizedLock.Lock() - if vm.txPoolStabilizedHead == h.Block.Hash() { - vm.txPoolStabilizedOk <- struct{}{} - vm.txPoolStabilizedHead = common.Hash{} - } - vm.txPoolStabilizedLock.Unlock() - } - } - }) + vm.shutdownSubmitChan = make(chan struct{}, 1) + vm.newMinedBlockSub = vm.chain.SubscribeNewMinedBlockEvent() + vm.shutdownWg.Add(1) + go ctx.Log.RecoverAndPanic(vm.awaitTxPoolStabilized) chain.Start() var lastAccepted *types.Block @@ -365,14 +365,14 @@ func (vm *VM) Initialize( var hash common.Hash if err = rlp.DecodeBytes(b, &hash); err == nil { if block := chain.GetBlockByHash(hash); block == nil { - vm.ctx.Log.Debug("lastAccepted block not found in chaindb") + log.Debug("lastAccepted block not found in chaindb") } else { lastAccepted = block } } } if lastAccepted == nil { - vm.ctx.Log.Debug("lastAccepted is unavailable, setting to the genesis block") + log.Debug("lastAccepted is unavailable, setting to the genesis block") lastAccepted = chain.GetGenesisBlock() } vm.lastAccepted = &Block{ @@ -381,24 +381,11 @@ func (vm *VM) Initialize( vm: vm, } vm.genesisHash = chain.GetGenesisBlock().Hash() - vm.ctx.Log.Info(fmt.Sprintf("lastAccepted = %s", vm.lastAccepted.ethBlock.Hash().Hex())) + log.Info(fmt.Sprintf("lastAccepted = %s", vm.lastAccepted.ethBlock.Hash().Hex())) // TODO: shutdown this go routine - go vm.ctx.Log.RecoverAndPanic(func() { - vm.txSubmitChan = vm.chain.GetTxSubmitCh() - for { - select { - case <-vm.txSubmitChan: - vm.ctx.Log.Verbo("New tx detected, trying to generate a block") - vm.tryBlockGen() - case <-vm.atomicTxSubmitChan: - vm.ctx.Log.Verbo("New atomic Tx detected, trying to generate a block") - vm.tryBlockGen() - case <-time.After(5 * time.Second): - vm.tryBlockGen() - } - } - }) + vm.shutdownWg.Add(1) + go vm.ctx.Log.RecoverAndPanic(vm.awaitSubmittedTxs) vm.codec = Codec return vm.fx.Initialize(vm) @@ -419,7 +406,10 @@ func (vm *VM) Shutdown() error { } vm.writeBackMetadata() + close(vm.txPoolStabilizedShutdownChan) + close(vm.shutdownSubmitChan) vm.chain.Stop() + vm.shutdownWg.Wait() return nil } @@ -438,7 +428,7 @@ func (vm *VM) BuildBlock() (snowman.Block, error) { vm.blockDelayTimer.SetTimeoutIn(minBlockTime) vm.bdlock.Unlock() - vm.ctx.Log.Debug("built block 0x%x", block.ID().Bytes()) + log.Debug(fmt.Sprintf("built block 0x%x", block.ID().Bytes())) // make sure Tx Pool is updated <-vm.txPoolStabilizedOk return block, nil @@ -622,7 +612,7 @@ func (vm *VM) getCachedStatus(blockID ids.ID) choices.Status { acceptedIDBytes, err := vm.acceptedDB.Get(heightKey) if err == nil { if acceptedID, err := ids.ToID(acceptedIDBytes); err != nil { - vm.ctx.Log.Error("snowman-eth: acceptedID bytes didn't match expected value: %s", err) + log.Error(fmt.Sprintf("snowman-eth: acceptedID bytes didn't match expected value: %s", err)) } else { if acceptedID.Equals(blockID) { vm.blockStatusCache.Put(blockID, choices.Accepted) @@ -637,7 +627,7 @@ func (vm *VM) getCachedStatus(blockID ids.ID) choices.Status { if status == choices.Accepted { err := vm.acceptedDB.Put(heightKey, blockID.Bytes()) if err != nil { - vm.ctx.Log.Error("snowman-eth: failed to write back acceptedID bytes: %s", err) + log.Error(fmt.Sprintf("snowman-eth: failed to write back acceptedID bytes: %s", err)) } tempBlock := wrappedBlk @@ -655,7 +645,7 @@ func (vm *VM) getCachedStatus(blockID ids.ID) choices.Status { } if err := vm.acceptedDB.Put(heightKey, parentID.Bytes()); err != nil { - vm.ctx.Log.Error("snowman-eth: failed to write back acceptedID bytes: %s", err) + log.Error(fmt.Sprintf("snowman-eth: failed to write back acceptedID bytes: %s", err)) } } } @@ -725,14 +715,58 @@ func (vm *VM) writeBackMetadata() { b, err := rlp.EncodeToBytes(vm.lastAccepted.ethBlock.Hash()) if err != nil { - vm.ctx.Log.Error("snowman-eth: error while writing back metadata") + log.Error("snowman-eth: error while writing back metadata") return } - vm.ctx.Log.Debug("writing back metadata") + log.Debug("writing back metadata") vm.chaindb.Put([]byte(lastAcceptedKey), b) atomic.StoreUint32(&vm.writingMetadata, 0) } +// awaitTxPoolStabilized waits for a txPoolHead channel event +// and notifies the VM when the tx pool has stabilized to the +// expected block hash +// Waits for signal to shutdown from txPoolStabilizedShutdownChan chan +func (vm *VM) awaitTxPoolStabilized() { + defer vm.shutdownWg.Done() + for { + select { + case e := <-vm.newMinedBlockSub.Chan(): + switch h := e.Data.(type) { + case core.NewMinedBlockEvent: + vm.txPoolStabilizedLock.Lock() + if vm.txPoolStabilizedHead == h.Block.Hash() { + vm.txPoolStabilizedOk <- struct{}{} + vm.txPoolStabilizedHead = common.Hash{} + } + vm.txPoolStabilizedLock.Unlock() + default: + } + case <-vm.txPoolStabilizedShutdownChan: + return + } + } +} + +func (vm *VM) awaitSubmittedTxs() { + defer vm.shutdownWg.Done() + vm.txSubmitChan = vm.chain.GetTxSubmitCh() + for { + select { + case <-vm.txSubmitChan: + log.Trace("New tx detected, trying to generate a block") + vm.tryBlockGen() + case <-vm.atomicTxSubmitChan: + log.Trace("New atomic Tx detected, trying to generate a block") + vm.tryBlockGen() + case <-time.After(5 * time.Second): + vm.tryBlockGen() + case <-vm.shutdownSubmitChan: + return + } + } +} + func (vm *VM) getLastAccepted() *Block { vm.metalock.Lock() defer vm.metalock.Unlock() |