updated blockpool

pull/532/head
obscuren 10 years ago
parent 4e181c5764
commit 843db4978e
  1. 51
      blockpool/blockpool.go
  2. 49
      blockpool/peers.go
  3. 15
      blockpool/section.go
  4. 3
      common/types.go
  5. 6
      core/types/block.go
  6. 6
      core/types/bloom9.go
  7. 36
      core/types/receipt.go
  8. 20
      state/log.go
  9. 55
      tests/blocktest.go
  10. 12
      vm/environment.go
  11. 4
      vm/vm.go

@ -1,12 +1,12 @@
package blockpool package blockpool
import ( import (
"bytes"
"fmt" "fmt"
"math/big" "math/big"
"sync" "sync"
"time" "time"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/core/types" "github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/errs" "github.com/ethereum/go-ethereum/errs"
ethlogger "github.com/ethereum/go-ethereum/logger" ethlogger "github.com/ethereum/go-ethereum/logger"
@ -101,7 +101,7 @@ func (self *Config) init() {
// node is the basic unit of the internal model of block chain/tree in the blockpool // node is the basic unit of the internal model of block chain/tree in the blockpool
type node struct { type node struct {
lock sync.RWMutex lock sync.RWMutex
hash []byte hash common.Hash
block *types.Block block *types.Block
hashBy string hashBy string
blockBy string blockBy string
@ -123,7 +123,7 @@ type BlockPool struct {
Config *Config Config *Config
// the minimal interface with blockchain // the minimal interface with blockchain
hasBlock func(hash []byte) bool hasBlock func(hash common.Hash) bool
insertChain func(types.Blocks) error insertChain func(types.Blocks) error
verifyPoW func(pow.Block) bool verifyPoW func(pow.Block) bool
@ -133,7 +133,7 @@ type BlockPool struct {
lock sync.RWMutex lock sync.RWMutex
chainLock sync.RWMutex chainLock sync.RWMutex
// alloc-easy pool of hash slices // alloc-easy pool of hash slices
hashSlicePool chan [][]byte hashSlicePool chan []common.Hash
status *status status *status
@ -144,7 +144,7 @@ type BlockPool struct {
// public constructor // public constructor
func New( func New(
hasBlock func(hash []byte) bool, hasBlock func(hash common.Hash) bool,
insertChain func(types.Blocks) error, insertChain func(types.Blocks) error,
verifyPoW func(pow.Block) bool, verifyPoW func(pow.Block) bool,
) *BlockPool { ) *BlockPool {
@ -176,7 +176,7 @@ func (self *BlockPool) Start() {
} }
self.Config.init() self.Config.init()
self.hashSlicePool = make(chan [][]byte, 150) self.hashSlicePool = make(chan []common.Hash, 150)
self.status = newStatus() self.status = newStatus()
self.quit = make(chan bool) self.quit = make(chan bool)
self.pool = make(map[string]*entry) self.pool = make(map[string]*entry)
@ -261,14 +261,13 @@ Peer info is currently not persisted across disconnects (or sessions)
*/ */
func (self *BlockPool) AddPeer( func (self *BlockPool) AddPeer(
td *big.Int, currentBlockHash []byte, td *big.Int, currentBlockHash common.Hash,
peerId string, peerId string,
requestBlockHashes func([]byte) error, requestBlockHashes func(common.Hash) error,
requestBlocks func([][]byte) error, requestBlocks func([]common.Hash) error,
peerError func(*errs.Error), peerError func(*errs.Error),
) (best bool) { ) (best bool) {
return self.peers.addPeer(td, currentBlockHash, peerId, requestBlockHashes, requestBlocks, peerError) return self.peers.addPeer(td, currentBlockHash, peerId, requestBlockHashes, requestBlocks, peerError)
} }
@ -289,7 +288,7 @@ launches all block request processes on each chain section
the first argument is an iterator function. Using this block hashes are decoded from the rlp message payload on demand. As a result, AddBlockHashes needs to run synchronously for one peer since the message is discarded if the caller thread returns. the first argument is an iterator function. Using this block hashes are decoded from the rlp message payload on demand. As a result, AddBlockHashes needs to run synchronously for one peer since the message is discarded if the caller thread returns.
*/ */
func (self *BlockPool) AddBlockHashes(next func() ([]byte, bool), peerId string) { func (self *BlockPool) AddBlockHashes(next func() (common.Hash, bool), peerId string) {
bestpeer, best := self.peers.getPeer(peerId) bestpeer, best := self.peers.getPeer(peerId)
if !best { if !best {
@ -306,7 +305,7 @@ func (self *BlockPool) AddBlockHashes(next func() ([]byte, bool), peerId string)
self.status.lock.Unlock() self.status.lock.Unlock()
var n int var n int
var hash []byte var hash common.Hash
var ok, headSection, peerswitch bool var ok, headSection, peerswitch bool
var sec, child, parent *section var sec, child, parent *section
var entry *entry var entry *entry
@ -318,7 +317,7 @@ func (self *BlockPool) AddBlockHashes(next func() ([]byte, bool), peerId string)
plog.Debugf("AddBlockHashes: peer <%s> starting from [%s] (peer head: %s)", peerId, hex(bestpeer.parentHash), hex(bestpeer.currentBlockHash)) plog.Debugf("AddBlockHashes: peer <%s> starting from [%s] (peer head: %s)", peerId, hex(bestpeer.parentHash), hex(bestpeer.currentBlockHash))
// first check if we are building the head section of a peer's chain // first check if we are building the head section of a peer's chain
if bytes.Equal(bestpeer.parentHash, hash) { if bestpeer.parentHash == hash {
if self.hasBlock(bestpeer.currentBlockHash) { if self.hasBlock(bestpeer.currentBlockHash) {
return return
} }
@ -561,7 +560,7 @@ func (self *BlockPool) AddBlock(block *types.Block, peerId string) {
entry := self.get(hash) entry := self.get(hash)
// a peer's current head block is appearing the first time // a peer's current head block is appearing the first time
if bytes.Equal(hash, sender.currentBlockHash) { if hash == sender.currentBlockHash {
if sender.currentBlock == nil { if sender.currentBlock == nil {
plog.Debugf("AddBlock: add head block %s for peer <%s> (head: %s)", hex(hash), peerId, hex(sender.currentBlockHash)) plog.Debugf("AddBlock: add head block %s for peer <%s> (head: %s)", hex(hash), peerId, hex(sender.currentBlockHash))
sender.setChainInfoFromBlock(block) sender.setChainInfoFromBlock(block)
@ -664,7 +663,7 @@ LOOP:
plog.DebugDetailf("activateChain: section [%s] activated by peer <%s>", sectionhex(sec), p.id) plog.DebugDetailf("activateChain: section [%s] activated by peer <%s>", sectionhex(sec), p.id)
sec.activate(p) sec.activate(p)
if i > 0 && connected != nil { if i > 0 && connected != nil {
connected[string(sec.top.hash)] = sec connected[sec.top.hash.Str()] = sec
} }
/* /*
we need to relink both complete and incomplete sections we need to relink both complete and incomplete sections
@ -696,7 +695,7 @@ LOOP:
// must run in separate go routine, otherwise // must run in separate go routine, otherwise
// switchpeer -> activateChain -> activate deadlocks on section process select and peers.lock // switchpeer -> activateChain -> activate deadlocks on section process select and peers.lock
func (self *BlockPool) requestBlocks(attempts int, hashes [][]byte) { func (self *BlockPool) requestBlocks(attempts int, hashes []common.Hash) {
self.wg.Add(1) self.wg.Add(1)
go func() { go func() {
self.peers.requestBlocks(attempts, hashes) self.peers.requestBlocks(attempts, hashes)
@ -718,16 +717,16 @@ func (self *BlockPool) getChild(sec *section) *section {
} }
// accessor and setter for entries in the pool // accessor and setter for entries in the pool
func (self *BlockPool) get(hash []byte) *entry { func (self *BlockPool) get(hash common.Hash) *entry {
self.lock.RLock() self.lock.RLock()
defer self.lock.RUnlock() defer self.lock.RUnlock()
return self.pool[string(hash)] return self.pool[hash.Str()]
} }
func (self *BlockPool) set(hash []byte, e *entry) { func (self *BlockPool) set(hash common.Hash, e *entry) {
self.lock.Lock() self.lock.Lock()
defer self.lock.Unlock() defer self.lock.Unlock()
self.pool[string(hash)] = e self.pool[hash.Str()] = e
} }
func (self *BlockPool) remove(sec *section) { func (self *BlockPool) remove(sec *section) {
@ -736,7 +735,7 @@ func (self *BlockPool) remove(sec *section) {
defer self.lock.Unlock() defer self.lock.Unlock()
for _, node := range sec.nodes { for _, node := range sec.nodes {
delete(self.pool, string(node.hash)) delete(self.pool, node.hash.Str())
} }
if sec.initialised && sec.poolRootIndex != 0 { if sec.initialised && sec.poolRootIndex != 0 {
self.status.lock.Lock() self.status.lock.Lock()
@ -745,17 +744,17 @@ func (self *BlockPool) remove(sec *section) {
} }
} }
func (self *BlockPool) getHashSlice() (s [][]byte) { func (self *BlockPool) getHashSlice() (s []common.Hash) {
select { select {
case s = <-self.hashSlicePool: case s = <-self.hashSlicePool:
default: default:
s = make([][]byte, self.Config.BlockBatchSize) s = make([]common.Hash, self.Config.BlockBatchSize)
} }
return return
} }
// Return returns a Client to the pool. // Return returns a Client to the pool.
func (self *BlockPool) putHashSlice(s [][]byte) { func (self *BlockPool) putHashSlice(s []common.Hash) {
if len(s) == self.Config.BlockBatchSize { if len(s) == self.Config.BlockBatchSize {
select { select {
case self.hashSlicePool <- s: case self.hashSlicePool <- s:
@ -765,8 +764,8 @@ func (self *BlockPool) putHashSlice(s [][]byte) {
} }
// pretty prints hash (byte array) with first 4 bytes in hex // pretty prints hash (byte array) with first 4 bytes in hex
func hex(hash []byte) (name string) { func hex(hash common.Hash) (name string) {
if hash == nil { if (hash == common.Hash{}) {
name = "" name = ""
} else { } else {
name = fmt.Sprintf("%x", hash[:4]) name = fmt.Sprintf("%x", hash[:4])

@ -1,16 +1,15 @@
package blockpool package blockpool
import ( import (
"bytes"
"math/big" "math/big"
"math/rand" "math/rand"
"sort" "sort"
"sync" "sync"
"time" "time"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/core/types" "github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/errs" "github.com/ethereum/go-ethereum/errs"
"github.com/ethereum/go-ethereum/common"
) )
type peer struct { type peer struct {
@ -18,20 +17,20 @@ type peer struct {
// last known blockchain status // last known blockchain status
td *big.Int td *big.Int
currentBlockHash []byte currentBlockHash common.Hash
currentBlock *types.Block currentBlock *types.Block
parentHash []byte parentHash common.Hash
headSection *section headSection *section
id string id string
// peer callbacks // peer callbacks
requestBlockHashes func([]byte) error requestBlockHashes func(common.Hash) error
requestBlocks func([][]byte) error requestBlocks func([]common.Hash) error
peerError func(*errs.Error) peerError func(*errs.Error)
errors *errs.Errors errors *errs.Errors
sections [][]byte sections []common.Hash
// channels to push new head block and head section for peer a // channels to push new head block and head section for peer a
currentBlockC chan *types.Block currentBlockC chan *types.Block
@ -66,10 +65,10 @@ type peers struct {
// peer constructor // peer constructor
func (self *peers) newPeer( func (self *peers) newPeer(
td *big.Int, td *big.Int,
currentBlockHash []byte, currentBlockHash common.Hash,
id string, id string,
requestBlockHashes func([]byte) error, requestBlockHashes func(common.Hash) error,
requestBlocks func([][]byte) error, requestBlocks func([]common.Hash) error,
peerError func(*errs.Error), peerError func(*errs.Error),
) (p *peer) { ) (p *peer) {
@ -107,7 +106,7 @@ func (self *peer) addError(code int, format string, params ...interface{}) {
self.peerError(err) self.peerError(err)
} }
func (self *peer) setChainInfo(td *big.Int, c []byte) { func (self *peer) setChainInfo(td *big.Int, c common.Hash) {
self.lock.Lock() self.lock.Lock()
defer self.lock.Unlock() defer self.lock.Unlock()
@ -115,7 +114,7 @@ func (self *peer) setChainInfo(td *big.Int, c []byte) {
self.currentBlockHash = c self.currentBlockHash = c
self.currentBlock = nil self.currentBlock = nil
self.parentHash = nil self.parentHash = common.Hash{}
self.headSection = nil self.headSection = nil
} }
@ -139,7 +138,7 @@ func (self *peer) setChainInfoFromBlock(block *types.Block) {
}() }()
} }
func (self *peers) requestBlocks(attempts int, hashes [][]byte) { func (self *peers) requestBlocks(attempts int, hashes []common.Hash) {
// distribute block request among known peers // distribute block request among known peers
self.lock.RLock() self.lock.RLock()
defer self.lock.RUnlock() defer self.lock.RUnlock()
@ -178,18 +177,18 @@ func (self *peers) requestBlocks(attempts int, hashes [][]byte) {
// returns true iff peer is promoted as best peer in the pool // returns true iff peer is promoted as best peer in the pool
func (self *peers) addPeer( func (self *peers) addPeer(
td *big.Int, td *big.Int,
currentBlockHash []byte, currentBlockHash common.Hash,
id string, id string,
requestBlockHashes func([]byte) error, requestBlockHashes func(common.Hash) error,
requestBlocks func([][]byte) error, requestBlocks func([]common.Hash) error,
peerError func(*errs.Error), peerError func(*errs.Error),
) (best bool) { ) (best bool) {
var previousBlockHash []byte var previousBlockHash common.Hash
self.lock.Lock() self.lock.Lock()
p, found := self.peers[id] p, found := self.peers[id]
if found { if found {
if !bytes.Equal(p.currentBlockHash, currentBlockHash) { if p.currentBlockHash != currentBlockHash {
previousBlockHash = p.currentBlockHash previousBlockHash = p.currentBlockHash
plog.Debugf("addPeer: Update peer <%s> with td %v and current block %s (was %v)", id, td, hex(currentBlockHash), hex(previousBlockHash)) plog.Debugf("addPeer: Update peer <%s> with td %v and current block %s (was %v)", id, td, hex(currentBlockHash), hex(previousBlockHash))
p.setChainInfo(td, currentBlockHash) p.setChainInfo(td, currentBlockHash)
@ -221,7 +220,7 @@ func (self *peers) addPeer(
// new block update for active current best peer -> request hashes // new block update for active current best peer -> request hashes
plog.Debugf("addPeer: <%s> already the best peer. Request new head section info from %s", id, hex(currentBlockHash)) plog.Debugf("addPeer: <%s> already the best peer. Request new head section info from %s", id, hex(currentBlockHash))
if previousBlockHash != nil { if (previousBlockHash != common.Hash{}) {
if entry := self.bp.get(previousBlockHash); entry != nil { if entry := self.bp.get(previousBlockHash); entry != nil {
p.headSectionC <- nil p.headSectionC <- nil
self.bp.activateChain(entry.section, p, nil) self.bp.activateChain(entry.section, p, nil)
@ -318,15 +317,15 @@ func (self *BlockPool) switchPeer(oldp, newp *peer) {
} }
var connected = make(map[string]*section) var connected = make(map[string]*section)
var sections [][]byte var sections []common.Hash
for _, hash := range newp.sections { for _, hash := range newp.sections {
plog.DebugDetailf("activate chain starting from section [%s]", hex(hash)) plog.DebugDetailf("activate chain starting from section [%s]", hex(hash))
// if section not connected (ie, top of a contiguous sequence of sections) // if section not connected (ie, top of a contiguous sequence of sections)
if connected[string(hash)] == nil { if connected[hash.Str()] == nil {
// if not deleted, then reread from pool (it can be orphaned top half of a split section) // if not deleted, then reread from pool (it can be orphaned top half of a split section)
if entry := self.get(hash); entry != nil { if entry := self.get(hash); entry != nil {
self.activateChain(entry.section, newp, connected) self.activateChain(entry.section, newp, connected)
connected[string(hash)] = entry.section connected[hash.Str()] = entry.section
sections = append(sections, hash) sections = append(sections, hash)
} }
} }
@ -396,7 +395,7 @@ func (self *peer) getCurrentBlock(currentBlock *types.Block) {
plog.DebugDetailf("HeadSection: <%s> head block %s found in blockpool", self.id, hex(self.currentBlockHash)) plog.DebugDetailf("HeadSection: <%s> head block %s found in blockpool", self.id, hex(self.currentBlockHash))
} else { } else {
plog.DebugDetailf("HeadSection: <%s> head block %s not found... requesting it", self.id, hex(self.currentBlockHash)) plog.DebugDetailf("HeadSection: <%s> head block %s not found... requesting it", self.id, hex(self.currentBlockHash))
self.requestBlocks([][]byte{self.currentBlockHash}) self.requestBlocks([]common.Hash{self.currentBlockHash})
self.blocksRequestTimer = time.After(self.bp.Config.BlocksRequestInterval) self.blocksRequestTimer = time.After(self.bp.Config.BlocksRequestInterval)
return return
} }
@ -427,9 +426,9 @@ func (self *peer) getBlockHashes() {
self.addError(ErrInvalidBlock, "%v", err) self.addError(ErrInvalidBlock, "%v", err)
self.bp.status.badPeers[self.id]++ self.bp.status.badPeers[self.id]++
} else { } else {
headKey := string(self.parentHash) headKey := self.parentHash.Str()
height := self.bp.status.chain[headKey] + 1 height := self.bp.status.chain[headKey] + 1
self.bp.status.chain[string(self.currentBlockHash)] = height self.bp.status.chain[self.currentBlockHash.Str()] = height
if height > self.bp.status.values.LongestChain { if height > self.bp.status.values.LongestChain {
self.bp.status.values.LongestChain = height self.bp.status.values.LongestChain = height
} }

@ -4,6 +4,7 @@ import (
"sync" "sync"
"time" "time"
"github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/core/types" "github.com/ethereum/go-ethereum/core/types"
) )
@ -27,9 +28,9 @@ type section struct {
nodes []*node nodes []*node
peer *peer peer *peer
parentHash []byte parentHash common.Hash
blockHashes [][]byte blockHashes []common.Hash
poolRootIndex int poolRootIndex int
@ -115,7 +116,7 @@ func (self *section) addSectionToBlockChain(p *peer) {
break break
} }
self.poolRootIndex-- self.poolRootIndex--
keys = append(keys, string(node.hash)) keys = append(keys, node.hash.Str())
blocks = append(blocks, block) blocks = append(blocks, block)
} }
@ -166,9 +167,9 @@ func (self *section) addSectionToBlockChain(p *peer) {
self.bp.status.lock.Lock() self.bp.status.lock.Lock()
if err == nil { if err == nil {
headKey := string(blocks[0].ParentHash()) headKey := blocks[0].ParentHash().Str()
height := self.bp.status.chain[headKey] + len(blocks) height := self.bp.status.chain[headKey] + len(blocks)
self.bp.status.chain[string(blocks[len(blocks)-1].Hash())] = height self.bp.status.chain[blocks[len(blocks)-1].Hash().Str()] = height
if height > self.bp.status.values.LongestChain { if height > self.bp.status.values.LongestChain {
self.bp.status.values.LongestChain = height self.bp.status.values.LongestChain = height
} }
@ -316,7 +317,7 @@ LOOP:
self.addSectionToBlockChain(self.peer) self.addSectionToBlockChain(self.peer)
} }
} else { } else {
if self.parentHash == nil && n == self.bottom { if (self.parentHash == common.Hash{}) && n == self.bottom {
self.parentHash = block.ParentHash() self.parentHash = block.ParentHash()
plog.DebugDetailf("[%s] got parent head block hash %s...checking", sectionhex(self), hex(self.parentHash)) plog.DebugDetailf("[%s] got parent head block hash %s...checking", sectionhex(self), hex(self.parentHash))
self.blockHashesRequest() self.blockHashesRequest()
@ -456,7 +457,7 @@ func (self *section) blockHashesRequest() {
// a demoted peer's fork will be chosen over the best peer's chain // a demoted peer's fork will be chosen over the best peer's chain
// because relinking the correct chain (activateChain) is overwritten here in // because relinking the correct chain (activateChain) is overwritten here in
// demoted peer's section process just before the section is put to idle mode // demoted peer's section process just before the section is put to idle mode
if self.parentHash != nil { if (self.parentHash != common.Hash{}) {
if parent := self.bp.get(self.parentHash); parent != nil { if parent := self.bp.get(self.parentHash); parent != nil {
parentSection = parent.section parentSection = parent.section
plog.DebugDetailf("[%s] blockHashesRequest: parent section [%s] linked\n", sectionhex(self), sectionhex(parentSection)) plog.DebugDetailf("[%s] blockHashesRequest: parent section [%s] linked\n", sectionhex(self), sectionhex(parentSection))

@ -24,6 +24,7 @@ func BytesToHash(b []byte) Hash {
} }
func StringToHash(s string) Hash { return BytesToHash([]byte(s)) } func StringToHash(s string) Hash { return BytesToHash([]byte(s)) }
func BigToHash(b *big.Int) Hash { return BytesToHash(b.Bytes()) } func BigToHash(b *big.Int) Hash { return BytesToHash(b.Bytes()) }
func HexToHash(s string) Hash { return BytesToHash(FromHex(s)) }
// Don't use the default 'String' method in case we want to overwrite // Don't use the default 'String' method in case we want to overwrite
@ -62,11 +63,13 @@ func BytesToAddress(b []byte) Address {
} }
func StringToAddress(s string) Address { return BytesToAddress([]byte(s)) } func StringToAddress(s string) Address { return BytesToAddress([]byte(s)) }
func BigToAddress(b *big.Int) Address { return BytesToAddress(b.Bytes()) } func BigToAddress(b *big.Int) Address { return BytesToAddress(b.Bytes()) }
func HexToAddress(s string) Address { return BytesToAddress(FromHex(s)) }
// Get the string representation of the underlying address // Get the string representation of the underlying address
func (a Address) Str() string { return string(a[:]) } func (a Address) Str() string { return string(a[:]) }
func (a Address) Bytes() []byte { return a[:] } func (a Address) Bytes() []byte { return a[:] }
func (a Address) Big() *big.Int { return Bytes2Big(a[:]) } func (a Address) Big() *big.Int { return Bytes2Big(a[:]) }
func (a Address) Hash() Hash { return BytesToHash(a[:]) }
// Sets the address to the value of b. If b is larger than len(a) it will panic // Sets the address to the value of b. If b is larger than len(a) it will panic
func (a *Address) SetBytes(b []byte) { func (a *Address) SetBytes(b []byte) {

@ -106,14 +106,14 @@ func NewBlock(parentHash common.Hash, coinbase common.Address, root common.Hash,
GasUsed: new(big.Int), GasUsed: new(big.Int),
GasLimit: new(big.Int), GasLimit: new(big.Int),
} }
header.setNonce(nonce) header.SetNonce(nonce)
block := &Block{header: header, Reward: new(big.Int)} block := &Block{header: header, Reward: new(big.Int)}
return block return block
} }
func (self *Header) setNonce(nonce uint64) { func (self *Header) SetNonce(nonce uint64) {
binary.BigEndian.PutUint64(self.Nonce[:], nonce) binary.BigEndian.PutUint64(self.Nonce[:], nonce)
} }
@ -203,7 +203,7 @@ func (self *Block) Nonce() uint64 {
return binary.BigEndian.Uint64(self.header.Nonce[:]) return binary.BigEndian.Uint64(self.header.Nonce[:])
} }
func (self *Block) SetNonce(nonce uint64) { func (self *Block) SetNonce(nonce uint64) {
self.header.setNonce(nonce) self.header.SetNonce(nonce)
} }
func (self *Block) Bloom() Bloom { return self.header.Bloom } func (self *Block) Bloom() Bloom { return self.header.Bloom }

@ -20,15 +20,15 @@ func CreateBloom(receipts Receipts) Bloom {
func LogsBloom(logs state.Logs) *big.Int { func LogsBloom(logs state.Logs) *big.Int {
bin := new(big.Int) bin := new(big.Int)
for _, log := range logs { for _, log := range logs {
data := make([][]byte, len(log.Topics())+1) data := make([]common.Hash, len(log.Topics())+1)
data[0] = log.Address() data[0] = log.Address().Hash()
for i, topic := range log.Topics() { for i, topic := range log.Topics() {
data[i+1] = topic data[i+1] = topic
} }
for _, b := range data { for _, b := range data {
bin.Or(bin, common.BigD(bloom9(crypto.Sha3(b)).Bytes())) bin.Or(bin, common.BigD(bloom9(crypto.Sha3(b[:])).Bytes()))
} }
} }

@ -3,9 +3,11 @@ package types
import ( import (
"bytes" "bytes"
"fmt" "fmt"
"io"
"math/big" "math/big"
"github.com/ethereum/go-ethereum/common" "github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/rlp"
"github.com/ethereum/go-ethereum/state" "github.com/ethereum/go-ethereum/state"
) )
@ -20,34 +22,26 @@ func NewReceipt(root []byte, cumalativeGasUsed *big.Int) *Receipt {
return &Receipt{PostState: common.CopyBytes(root), CumulativeGasUsed: new(big.Int).Set(cumalativeGasUsed)} return &Receipt{PostState: common.CopyBytes(root), CumulativeGasUsed: new(big.Int).Set(cumalativeGasUsed)}
} }
func NewRecieptFromValue(val *common.Value) *Receipt {
r := &Receipt{}
r.RlpValueDecode(val)
return r
}
func (self *Receipt) SetLogs(logs state.Logs) { func (self *Receipt) SetLogs(logs state.Logs) {
self.logs = logs self.logs = logs
} }
func (self *Receipt) RlpValueDecode(decoder *common.Value) { func (self *Receipt) EncodeRLP(w io.Writer) error {
self.PostState = decoder.Get(0).Bytes() return rlp.Encode(w, []interface{}{self.PostState, self.CumulativeGasUsed, self.Bloom, self.logs})
self.CumulativeGasUsed = decoder.Get(1).BigInt()
self.Bloom = decoder.Get(2).Bytes()
it := decoder.Get(3).NewIterator()
for it.Next() {
self.logs = append(self.logs, state.NewLogFromValue(it.Value()))
}
} }
/*
func (self *Receipt) RlpData() interface{} { func (self *Receipt) RlpData() interface{} {
return []interface{}{self.PostState, self.CumulativeGasUsed, self.Bloom, self.logs.RlpData()} return []interface{}{self.PostState, self.CumulativeGasUsed, self.Bloom, self.logs.RlpData()}
} }
*/
func (self *Receipt) RlpEncode() []byte { func (self *Receipt) RlpEncode() []byte {
return common.Encode(self.RlpData()) bytes, err := rlp.EncodeToBytes(self)
if err != nil {
fmt.Println("TMP -- RECEIPT ENCODE ERROR", err)
}
return bytes
} }
func (self *Receipt) Cmp(other *Receipt) bool { func (self *Receipt) Cmp(other *Receipt) bool {
@ -64,6 +58,7 @@ func (self *Receipt) String() string {
type Receipts []*Receipt type Receipts []*Receipt
/*
func (self Receipts) RlpData() interface{} { func (self Receipts) RlpData() interface{} {
data := make([]interface{}, len(self)) data := make([]interface{}, len(self))
for i, receipt := range self { for i, receipt := range self {
@ -72,9 +67,14 @@ func (self Receipts) RlpData() interface{} {
return data return data
} }
*/
func (self Receipts) RlpEncode() []byte { func (self Receipts) RlpEncode() []byte {
return common.Encode(self.RlpData()) bytes, err := rlp.EncodeToBytes(self)
if err != nil {
fmt.Println("TMP -- RECEIPTS ENCODE ERROR", err)
}
return bytes
} }
func (self Receipts) Len() int { return len(self) } func (self Receipts) Len() int { return len(self) }

@ -2,15 +2,15 @@ package state
import ( import (
"fmt" "fmt"
"io"
"github.com/ethereum/go-ethereum/common" "github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/rlp"
) )
type Log interface { type Log interface {
common.RlpEncodable
Address() common.Address Address() common.Address
Topics() [][]byte Topics() []common.Hash
Data() []byte Data() []byte
Number() uint64 Number() uint64
@ -18,12 +18,12 @@ type Log interface {
type StateLog struct { type StateLog struct {
address common.Address address common.Address
topics [][]byte topics []common.Hash
data []byte data []byte
number uint64 number uint64
} }
func NewLog(address common.Address, topics [][]byte, data []byte, number uint64) *StateLog { func NewLog(address common.Address, topics []common.Hash, data []byte, number uint64) *StateLog {
return &StateLog{address, topics, data, number} return &StateLog{address, topics, data, number}
} }
@ -31,7 +31,7 @@ func (self *StateLog) Address() common.Address {
return self.address return self.address
} }
func (self *StateLog) Topics() [][]byte { func (self *StateLog) Topics() []common.Hash {
return self.topics return self.topics
} }
@ -63,9 +63,15 @@ func NewLogFromValue(decoder *common.Value) *StateLog {
} }
*/ */
func (self *StateLog) EncodeRLP(w io.Writer) error {
return rlp.Encode(w, []interface{}{self.address, self.topics, self.data})
}
/*
func (self *StateLog) RlpData() interface{} { func (self *StateLog) RlpData() interface{} {
return []interface{}{self.address, common.ByteSliceToInterface(self.topics), self.data} return []interface{}{self.address, common.ByteSliceToInterface(self.topics), self.data}
} }
*/
func (self *StateLog) String() string { func (self *StateLog) String() string {
return fmt.Sprintf(`log: %x %x %x`, self.address, self.topics, self.data) return fmt.Sprintf(`log: %x %x %x`, self.address, self.topics, self.data)
@ -73,6 +79,7 @@ func (self *StateLog) String() string {
type Logs []Log type Logs []Log
/*
func (self Logs) RlpData() interface{} { func (self Logs) RlpData() interface{} {
data := make([]interface{}, len(self)) data := make([]interface{}, len(self))
for i, log := range self { for i, log := range self {
@ -81,6 +88,7 @@ func (self Logs) RlpData() interface{} {
return data return data
} }
*/
func (self Logs) String() (ret string) { func (self Logs) String() (ret string) {
for _, log := range self { for _, log := range self {

@ -12,8 +12,9 @@ import (
"strconv" "strconv"
"strings" "strings"
"github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/common" "github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/core"
"github.com/ethereum/go-ethereum/core/types"
"github.com/ethereum/go-ethereum/rlp" "github.com/ethereum/go-ethereum/rlp"
"github.com/ethereum/go-ethereum/state" "github.com/ethereum/go-ethereum/state"
) )
@ -101,12 +102,12 @@ func (t *BlockTest) InsertPreState(db common.Database) error {
statedb := state.New(nil, db) statedb := state.New(nil, db)
for addrString, acct := range t.preAccounts { for addrString, acct := range t.preAccounts {
// XXX: is is worth it checking for errors here? // XXX: is is worth it checking for errors here?
addr, _ := hex.DecodeString(addrString) //addr, _ := hex.DecodeString(addrString)
code, _ := hex.DecodeString(strings.TrimPrefix(acct.Code, "0x")) code, _ := hex.DecodeString(strings.TrimPrefix(acct.Code, "0x"))
balance, _ := new(big.Int).SetString(acct.Balance, 0) balance, _ := new(big.Int).SetString(acct.Balance, 0)
nonce, _ := strconv.ParseUint(acct.Nonce, 16, 64) nonce, _ := strconv.ParseUint(acct.Nonce, 16, 64)
obj := statedb.NewStateObject(addr) obj := statedb.NewStateObject(common.HexToAddress(addrString))
obj.SetCode(code) obj.SetCode(code)
obj.SetBalance(balance) obj.SetBalance(balance)
obj.SetNonce(nonce) obj.SetNonce(nonce)
@ -119,7 +120,7 @@ func (t *BlockTest) InsertPreState(db common.Database) error {
// sync trie to disk // sync trie to disk
statedb.Sync() statedb.Sync()
if !bytes.Equal(t.Genesis.Root(), statedb.Root()) { if !bytes.Equal(t.Genesis.Root().Bytes(), statedb.Root()) {
return errors.New("computed state root does not match genesis block") return errors.New("computed state root does not match genesis block")
} }
return nil return nil
@ -153,23 +154,25 @@ func mustConvertGenesis(testGenesis btHeader) *types.Block {
func mustConvertHeader(in btHeader) *types.Header { func mustConvertHeader(in btHeader) *types.Header {
// hex decode these fields // hex decode these fields
return &types.Header{ header := &types.Header{
//SeedHash: mustConvertBytes(in.SeedHash), //SeedHash: mustConvertBytes(in.SeedHash),
MixDigest: mustConvertBytes(in.MixHash), MixDigest: mustConvertHash(in.MixHash),
Bloom: mustConvertBytes(in.Bloom), Bloom: mustConvertBloom(in.Bloom),
ReceiptHash: mustConvertBytes(in.ReceiptTrie), ReceiptHash: mustConvertHash(in.ReceiptTrie),
TxHash: mustConvertBytes(in.TransactionsTrie), TxHash: mustConvertHash(in.TransactionsTrie),
Root: mustConvertBytes(in.StateRoot), Root: mustConvertHash(in.StateRoot),
Coinbase: mustConvertBytes(in.Coinbase), Coinbase: mustConvertAddress(in.Coinbase),
UncleHash: mustConvertBytes(in.UncleHash), UncleHash: mustConvertHash(in.UncleHash),
ParentHash: mustConvertBytes(in.ParentHash), ParentHash: mustConvertHash(in.ParentHash),
Nonce: mustConvertBytes(in.Nonce),
Extra: string(mustConvertBytes(in.ExtraData)), Extra: string(mustConvertBytes(in.ExtraData)),
GasUsed: mustConvertBigInt10(in.GasUsed), GasUsed: mustConvertBigInt10(in.GasUsed),
GasLimit: mustConvertBigInt10(in.GasLimit), GasLimit: mustConvertBigInt10(in.GasLimit),
Difficulty: mustConvertBigInt10(in.Difficulty), Difficulty: mustConvertBigInt10(in.Difficulty),
Time: mustConvertUint(in.Timestamp), Time: mustConvertUint(in.Timestamp),
} }
// XXX cheats? :-)
header.SetNonce(common.BytesToHash(mustConvertBytes(in.Nonce)).Big().Uint64())
return header
} }
func mustConvertBlocks(testBlocks []btBlock) []*types.Block { func mustConvertBlocks(testBlocks []btBlock) []*types.Block {
@ -193,6 +196,30 @@ func mustConvertBytes(in string) []byte {
return out return out
} }
func mustConvertHash(in string) common.Hash {
out, err := hex.DecodeString(strings.TrimPrefix(in, "0x"))
if err != nil {
panic(fmt.Errorf("invalid hex: %q", in))
}
return common.BytesToHash(out)
}
func mustConvertAddress(in string) common.Address {
out, err := hex.DecodeString(strings.TrimPrefix(in, "0x"))
if err != nil {
panic(fmt.Errorf("invalid hex: %q", in))
}
return common.BytesToAddress(out)
}
func mustConvertBloom(in string) core.Bloom {
out, err := hex.DecodeString(strings.TrimPrefix(in, "0x"))
if err != nil {
panic(fmt.Errorf("invalid hex: %q", in))
}
return core.BytesToBloom(out)
}
func mustConvertBigInt10(in string) *big.Int { func mustConvertBigInt10(in string) *big.Int {
out, ok := new(big.Int).SetString(in, 10) out, ok := new(big.Int).SetString(in, 10)
if !ok { if !ok {

@ -3,9 +3,11 @@ package vm
import ( import (
"errors" "errors"
"fmt" "fmt"
"io"
"math/big" "math/big"
"github.com/ethereum/go-ethereum/common" "github.com/ethereum/go-ethereum/common"
"github.com/ethereum/go-ethereum/rlp"
"github.com/ethereum/go-ethereum/state" "github.com/ethereum/go-ethereum/state"
) )
@ -54,7 +56,7 @@ func Transfer(from, to Account, amount *big.Int) error {
type Log struct { type Log struct {
address common.Address address common.Address
topics [][]byte topics []common.Hash
data []byte data []byte
log uint64 log uint64
} }
@ -63,7 +65,7 @@ func (self *Log) Address() common.Address {
return self.address return self.address
} }
func (self *Log) Topics() [][]byte { func (self *Log) Topics() []common.Hash {
return self.topics return self.topics
} }
@ -75,9 +77,15 @@ func (self *Log) Number() uint64 {
return self.log return self.log
} }
func (self *Log) EncodeRLP(w io.Writer) error {
return rlp.Encode(w, []interface{}{self.address, self.topics, self.data})
}
/*
func (self *Log) RlpData() interface{} { func (self *Log) RlpData() interface{} {
return []interface{}{self.address, common.ByteSliceToInterface(self.topics), self.data} return []interface{}{self.address, common.ByteSliceToInterface(self.topics), self.data}
} }
*/
func (self *Log) String() string { func (self *Log) String() string {
return fmt.Sprintf("[A=%x T=%x D=%x]", self.address, self.topics, self.data) return fmt.Sprintf("[A=%x T=%x D=%x]", self.address, self.topics, self.data)

@ -560,10 +560,10 @@ func (self *Vm) Run(context *Context, callData []byte) (ret []byte, err error) {
self.Printf(" => [%d]", n) self.Printf(" => [%d]", n)
case LOG0, LOG1, LOG2, LOG3, LOG4: case LOG0, LOG1, LOG2, LOG3, LOG4:
n := int(op - LOG0) n := int(op - LOG0)
topics := make([][]byte, n) topics := make([]common.Hash, n)
mStart, mSize := stack.pop(), stack.pop() mStart, mSize := stack.pop(), stack.pop()
for i := 0; i < n; i++ { for i := 0; i < n; i++ {
topics[i] = common.LeftPadBytes(stack.pop().Bytes(), 32) topics[i] = common.BigToHash(stack.pop()) //common.LeftPadBytes(stack.pop().Bytes(), 32)
} }
data := mem.Get(mStart.Int64(), mSize.Int64()) data := mem.Get(mStart.Int64(), mSize.Int64())

Loading…
Cancel
Save