Browse Source

rpc: websocket events testing

pull/52/head
Ethan Buchman 10 years ago
parent
commit
2e918e8c0b
9 changed files with 416 additions and 116 deletions
  1. +19
    -16
      consensus/state.go
  2. +2
    -2
      mempool/mempool.go
  3. +0
    -35
      rpc/test/client_rpc_test.go
  4. +327
    -0
      rpc/test/client_ws_test.go
  5. +34
    -3
      rpc/test/helpers.go
  6. +1
    -37
      rpc/test/tests.go
  7. +24
    -22
      state/execution.go
  8. +1
    -1
      state/state.go
  9. +8
    -0
      vm/vm.go

+ 19
- 16
consensus/state.go View File

@ -73,11 +73,14 @@ import (
) )
const ( const (
roundDuration0 = 10 * time.Second // The first round is 60 seconds long.
roundDurationDelta = 3 * time.Second // Each successive round lasts 15 seconds longer.
roundDeadlinePrevote = float64(1.0 / 3.0) // When the prevote is due. roundDeadlinePrevote = float64(1.0 / 3.0) // When the prevote is due.
roundDeadlinePrecommit = float64(2.0 / 3.0) // When the precommit vote is due. roundDeadlinePrecommit = float64(2.0 / 3.0) // When the precommit vote is due.
newHeightDelta = roundDuration0 / 3 // The time to wait between commitTime and startTime of next consensus rounds.
)
var (
RoundDuration0 = 10 * time.Second // The first round is 60 seconds long.
RoundDurationDelta = 3 * time.Second // Each successive round lasts 15 seconds longer.
newHeightDelta = RoundDuration0 / 3 // The time to wait between commitTime and startTime of next consensus rounds.
) )
var ( var (
@ -318,14 +321,14 @@ func (cs *ConsensusState) stepTransitionRoutine() {
// NOTE: We can push directly to runActionCh because // NOTE: We can push directly to runActionCh because
// we're running in a separate goroutine, which avoids deadlocks. // we're running in a separate goroutine, which avoids deadlocks.
rs := cs.getRoundState() rs := cs.getRoundState()
round, roundStartTime, roundDuration, _, elapsedRatio := calcRoundInfo(rs.StartTime)
round, roundStartTime, RoundDuration, _, elapsedRatio := calcRoundInfo(rs.StartTime)
log.Debug("Scheduling next action", "height", rs.Height, "round", round, "step", rs.Step, "roundStartTime", roundStartTime, "elapsedRatio", elapsedRatio) log.Debug("Scheduling next action", "height", rs.Height, "round", round, "step", rs.Step, "roundStartTime", roundStartTime, "elapsedRatio", elapsedRatio)
switch rs.Step { switch rs.Step {
case RoundStepNewHeight: case RoundStepNewHeight:
// We should run RoundActionPropose when rs.StartTime passes. // We should run RoundActionPropose when rs.StartTime passes.
if elapsedRatio < 0 { if elapsedRatio < 0 {
// startTime is in the future. // startTime is in the future.
time.Sleep(time.Duration((-1.0 * elapsedRatio) * float64(roundDuration)))
time.Sleep(time.Duration((-1.0 * elapsedRatio) * float64(RoundDuration)))
} }
cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPropose} cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPropose}
case RoundStepNewRound: case RoundStepNewRound:
@ -333,15 +336,15 @@ func (cs *ConsensusState) stepTransitionRoutine() {
cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPropose} cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPropose}
case RoundStepPropose: case RoundStepPropose:
// Wake up when it's time to vote. // Wake up when it's time to vote.
time.Sleep(time.Duration((roundDeadlinePrevote - elapsedRatio) * float64(roundDuration)))
time.Sleep(time.Duration((roundDeadlinePrevote - elapsedRatio) * float64(RoundDuration)))
cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPrevote} cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPrevote}
case RoundStepPrevote: case RoundStepPrevote:
// Wake up when it's time to precommit. // Wake up when it's time to precommit.
time.Sleep(time.Duration((roundDeadlinePrecommit - elapsedRatio) * float64(roundDuration)))
time.Sleep(time.Duration((roundDeadlinePrecommit - elapsedRatio) * float64(RoundDuration)))
cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPrecommit} cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionPrecommit}
case RoundStepPrecommit: case RoundStepPrecommit:
// Wake up when the round is over. // Wake up when the round is over.
time.Sleep(time.Duration((1.0 - elapsedRatio) * float64(roundDuration)))
time.Sleep(time.Duration((1.0 - elapsedRatio) * float64(RoundDuration)))
cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionTryCommit} cs.runActionCh <- RoundAction{rs.Height, rs.Round, RoundActionTryCommit}
case RoundStepCommit: case RoundStepCommit:
// There's nothing to scheudle, we're waiting for // There's nothing to scheudle, we're waiting for
@ -1122,13 +1125,13 @@ func (cs *ConsensusState) SetEventSwitch(evsw *events.EventSwitch) {
// total duration of given round // total duration of given round
func calcRoundDuration(round uint) time.Duration { func calcRoundDuration(round uint) time.Duration {
return roundDuration0 + roundDurationDelta*time.Duration(round)
return RoundDuration0 + RoundDurationDelta*time.Duration(round)
} }
// startTime is when round zero started. // startTime is when round zero started.
func calcRoundStartTime(round uint, startTime time.Time) time.Time { func calcRoundStartTime(round uint, startTime time.Time) time.Time {
return startTime.Add(roundDuration0*time.Duration(round) +
roundDurationDelta*(time.Duration((int64(round)*int64(round)-int64(round))/2)))
return startTime.Add(RoundDuration0*time.Duration(round) +
RoundDurationDelta*(time.Duration((int64(round)*int64(round)-int64(round))/2)))
} }
// calculates the current round given startTime of round zero. // calculates the current round given startTime of round zero.
@ -1142,8 +1145,8 @@ func calcRound(startTime time.Time) uint {
// D_delta * R^2 + (2D_0 - D_delta) * R + 2(Start - Now) <= 0. // D_delta * R^2 + (2D_0 - D_delta) * R + 2(Start - Now) <= 0.
// AR^2 + BR + C <= 0; A = D_delta, B = (2_D0 - D_delta), C = 2(Start - Now). // AR^2 + BR + C <= 0; A = D_delta, B = (2_D0 - D_delta), C = 2(Start - Now).
// R = Floor((-B + Sqrt(B^2 - 4AC))/2A) // R = Floor((-B + Sqrt(B^2 - 4AC))/2A)
A := float64(roundDurationDelta)
B := 2.0*float64(roundDuration0) - float64(roundDurationDelta)
A := float64(RoundDurationDelta)
B := 2.0*float64(RoundDuration0) - float64(RoundDurationDelta)
C := 2.0 * float64(startTime.Sub(now)) C := 2.0 * float64(startTime.Sub(now))
R := math.Floor((-B + math.Sqrt(B*B-4.0*A*C)) / (2 * A)) R := math.Floor((-B + math.Sqrt(B*B-4.0*A*C)) / (2 * A))
if math.IsNaN(R) { if math.IsNaN(R) {
@ -1160,12 +1163,12 @@ func calcRound(startTime time.Time) uint {
// convenience // convenience
// NOTE: elapsedRatio can be negative if startTime is in the future. // NOTE: elapsedRatio can be negative if startTime is in the future.
func calcRoundInfo(startTime time.Time) (round uint, roundStartTime time.Time, roundDuration time.Duration,
func calcRoundInfo(startTime time.Time) (round uint, roundStartTime time.Time, RoundDuration time.Duration,
roundElapsed time.Duration, elapsedRatio float64) { roundElapsed time.Duration, elapsedRatio float64) {
round = calcRound(startTime) round = calcRound(startTime)
roundStartTime = calcRoundStartTime(round, startTime) roundStartTime = calcRoundStartTime(round, startTime)
roundDuration = calcRoundDuration(round)
RoundDuration = calcRoundDuration(round)
roundElapsed = time.Now().Sub(roundStartTime) roundElapsed = time.Now().Sub(roundStartTime)
elapsedRatio = float64(roundElapsed) / float64(roundDuration)
elapsedRatio = float64(roundElapsed) / float64(RoundDuration)
return return
} }

+ 2
- 2
mempool/mempool.go View File

@ -42,7 +42,7 @@ func (mem *Mempool) GetCache() *sm.BlockCache {
func (mem *Mempool) AddTx(tx types.Tx) (err error) { func (mem *Mempool) AddTx(tx types.Tx) (err error) {
mem.mtx.Lock() mem.mtx.Lock()
defer mem.mtx.Unlock() defer mem.mtx.Unlock()
err = sm.ExecTx(mem.cache, tx, false)
err = sm.ExecTx(mem.cache, tx, false, false)
if err != nil { if err != nil {
log.Debug("AddTx() error", "tx", tx, "error", err) log.Debug("AddTx() error", "tx", tx, "error", err)
return err return err
@ -93,7 +93,7 @@ func (mem *Mempool) ResetForBlockAndState(block *types.Block, state *sm.State) {
// Next, filter all txs that aren't valid given new state. // Next, filter all txs that aren't valid given new state.
validTxs := []types.Tx{} validTxs := []types.Tx{}
for _, tx := range txs { for _, tx := range txs {
err := sm.ExecTx(mem.cache, tx, false)
err := sm.ExecTx(mem.cache, tx, false, false)
if err == nil { if err == nil {
log.Debug("Filter in, valid", "tx", tx) log.Debug("Filter in, valid", "tx", tx)
validTxs = append(validTxs, tx) validTxs = append(validTxs, tx)


+ 0
- 35
rpc/test/client_rpc_test.go View File

@ -1,10 +1,6 @@
package rpc package rpc
import ( import (
"fmt"
"github.com/gorilla/websocket"
"github.com/tendermint/tendermint/rpc"
"net/http"
"testing" "testing"
) )
@ -77,34 +73,3 @@ func TestJSONCallCode(t *testing.T) {
func TestJSONCallContract(t *testing.T) { func TestJSONCallContract(t *testing.T) {
testCall(t, "JSONRPC") testCall(t, "JSONRPC")
} }
//--------------------------------------------------------------------------------
// Test the websocket client
func TestWSConnect(t *testing.T) {
dialer := websocket.DefaultDialer
rHeader := http.Header{}
_, r, err := dialer.Dial(websocketAddr, rHeader)
fmt.Println("respoinse:", r)
if err != nil {
t.Fatal(err)
}
}
func TestWSSubscribe(t *testing.T) {
dialer := websocket.DefaultDialer
rHeader := http.Header{}
con, _, err := dialer.Dial(websocketAddr, rHeader)
if err != nil {
t.Fatal(err)
}
err = con.WriteJSON(rpc.WSRequest{
Type: "subscribe",
Event: "newblock",
})
if err != nil {
t.Fatal(err)
}
typ, p, err := con.ReadMessage()
fmt.Println("RESPONSE:", typ, string(p), err)
}

+ 327
- 0
rpc/test/client_ws_test.go View File

@ -0,0 +1,327 @@
package rpc
import (
"bytes"
"encoding/hex"
"fmt"
"github.com/gorilla/websocket"
"github.com/tendermint/tendermint/binary"
"github.com/tendermint/tendermint/rpc"
"github.com/tendermint/tendermint/types"
"net/http"
"testing"
"time"
)
//--------------------------------------------------------------------------------
// Utilities for testing the websocket service
// create a new connection
func newWSCon(t *testing.T) *websocket.Conn {
dialer := websocket.DefaultDialer
rHeader := http.Header{}
con, r, err := dialer.Dial(websocketAddr, rHeader)
fmt.Println("response", r)
if err != nil {
t.Fatal(err)
}
return con
}
// subscribe to an event
func subscribe(t *testing.T, con *websocket.Conn, eventid string) {
err := con.WriteJSON(rpc.WSRequest{
Type: "subscribe",
Event: eventid,
})
if err != nil {
t.Fatal(err)
}
}
// unsubscribe from an event
func unsubscribe(t *testing.T, con *websocket.Conn, eventid string) {
err := con.WriteJSON(rpc.WSRequest{
Type: "unsubscribe",
Event: eventid,
})
if err != nil {
t.Fatal(err)
}
}
// wait for an event, do things that might trigger events, and check them when they are received
func waitForEvent(t *testing.T, con *websocket.Conn, eventid string, dieOnTimeout bool, f func(), check func(string, []byte) error) {
// go routine to wait for webscoket msg
gch := make(chan []byte)
ech := make(chan error)
go func() {
typ, p, err := con.ReadMessage()
fmt.Println("RESPONSE:", typ, string(p), err)
if err != nil {
ech <- err
} else {
gch <- p
}
}()
// do stuff (transactions)
f()
// if the event is not received in 20 seconds, die
ticker := time.Tick(10 * time.Second)
select {
case <-ticker:
if dieOnTimeout {
con.Close()
t.Fatalf("%s event was not received in time", eventid)
}
// else that's great, we didn't hear the event
case p := <-gch:
if dieOnTimeout {
// message was received and expected
// run the check
err := check(eventid, p)
if err != nil {
t.Fatal(err)
}
} else {
con.Close()
t.Fatalf("%s event was not expected", eventid)
}
case err := <-ech:
t.Fatal(err)
}
}
func unmarshalResponseNewBlock(b []byte) (*types.Block, error) {
// unmarshall and assert somethings
var response struct {
Event string
Data *types.Block
Error string
}
var err error
binary.ReadJSON(&response, b, &err)
if err != nil {
return nil, err
}
if response.Error != "" {
return nil, fmt.Errorf(response.Error)
}
block := response.Data
return block, nil
}
//--------------------------------------------------------------------------------
// Test the websocket service
// make a simple connection to the server
func TestWSConnect(t *testing.T) {
con := newWSCon(t)
con.Close()
}
// receive a new block message
func _TestWSNewBlock(t *testing.T) {
con := newWSCon(t)
eid := types.EventStringNewBlock()
subscribe(t, con, eid)
defer func() {
unsubscribe(t, con, eid)
con.Close()
}()
waitForEvent(t, con, eid, true, func() {}, func(eid string, b []byte) error {
fmt.Println("Check:", string(b))
return nil
})
}
// receive a few new block messages in a row, with increasing height
func TestWSBlockchainGrowth(t *testing.T) {
con := newWSCon(t)
eid := types.EventStringNewBlock()
subscribe(t, con, eid)
defer func() {
unsubscribe(t, con, eid)
con.Close()
}()
var initBlockN uint
for i := 0; i < 2; i++ {
waitForEvent(t, con, eid, true, func() {}, func(eid string, b []byte) error {
block, err := unmarshalResponseNewBlock(b)
if err != nil {
return err
}
if i == 0 {
initBlockN = block.Header.Height
} else {
if block.Header.Height != initBlockN+uint(i) {
return fmt.Errorf("Expected block %d, got block %d", i, block.Header.Height)
}
}
return nil
})
}
}
// send a transaction and validate the events from listening for both sender and receiver
func TestWSSend(t *testing.T) {
toAddr := []byte{20, 143, 25, 63, 16, 177, 83, 29, 91, 91, 54, 23, 233, 46, 190, 121, 122, 34, 86, 54}
amt := uint64(100)
con := newWSCon(t)
eidInput := types.EventStringAccInput(byteAddr)
eidOutput := types.EventStringAccOutput(toAddr)
subscribe(t, con, eidInput)
subscribe(t, con, eidOutput)
defer func() {
unsubscribe(t, con, eidInput)
unsubscribe(t, con, eidOutput)
con.Close()
}()
checkerFunc := func(eid string, b []byte) error {
// unmarshal and assert correctness
var response struct {
Event string
Data types.SendTx
Error string
}
var err error
binary.ReadJSON(&response, b, &err)
if err != nil {
return err
}
if response.Error != "" {
return fmt.Errorf(response.Error)
}
if eid != response.Event {
return fmt.Errorf("Eventid is not correct. Got %s, expected %s", response.Event, eid)
}
tx := response.Data
if bytes.Compare(tx.Inputs[0].Address, byteAddr) != 0 {
return fmt.Errorf("Senders do not match up! Got %x, expected %x", tx.Inputs[0].Address, byteAddr)
}
if tx.Inputs[0].Amount != amt {
return fmt.Errorf("Amt does not match up! Got %d, expected %d", tx.Inputs[0].Amount, amt)
}
if bytes.Compare(tx.Outputs[0].Address, toAddr) != 0 {
return fmt.Errorf("Receivers do not match up! Got %x, expected %x", tx.Outputs[0].Address, byteAddr)
}
return nil
}
waitForEvent(t, con, eidInput, true, func() {
broadcastTx(t, "JSONRPC", byteAddr, toAddr, nil, byteKey, amt, 0, 0)
}, checkerFunc)
waitForEvent(t, con, eidOutput, true, func() {}, checkerFunc)
}
// ensure events are only fired once for a given transaction
func TestWSDoubleFire(t *testing.T) {
con := newWSCon(t)
eid := types.EventStringAccInput(byteAddr)
subscribe(t, con, eid)
defer func() {
unsubscribe(t, con, eid)
con.Close()
}()
amt := uint64(100)
toAddr := []byte{20, 143, 25, 63, 16, 177, 83, 29, 91, 91, 54, 23, 233, 46, 190, 121, 122, 34, 86, 54}
// broadcast the transaction, wait to hear about it
waitForEvent(t, con, eid, true, func() {
broadcastTx(t, "JSONRPC", byteAddr, toAddr, nil, byteKey, amt, 0, 0)
}, func(eid string, b []byte) error {
return nil
})
// but make sure we don't hear about it twice
waitForEvent(t, con, eid, false, func() {
}, func(eid string, b []byte) error {
return nil
})
}
// create a contract and send it a msg, validate the return
func TestWSCall(t *testing.T) {
byteAddr, _ := hex.DecodeString(userAddr)
con := newWSCon(t)
eid := types.EventStringAccInput(byteAddr)
subscribe(t, con, eid)
defer func() {
unsubscribe(t, con, eid)
con.Close()
}()
amt := uint64(10000)
code, returnCode, returnVal := simpleCallContract()
var contractAddr []byte
// wait for the contract to be created
waitForEvent(t, con, eid, true, func() {
_, receipt := broadcastTx(t, "JSONRPC", byteAddr, nil, code, byteKey, amt, 1000, 1000)
contractAddr = receipt.ContractAddr
}, func(eid string, b []byte) error {
// unmarshall and assert somethings
var response struct {
Event string
Data struct {
Tx types.CallTx
Return []byte
Exception string
}
Error string
}
var err error
binary.ReadJSON(&response, b, &err)
if err != nil {
return err
}
if response.Error != "" {
return fmt.Errorf(response.Error)
}
if response.Data.Exception != "" {
return fmt.Errorf(response.Data.Exception)
}
tx := response.Data.Tx
if bytes.Compare(tx.Input.Address, byteAddr) != 0 {
return fmt.Errorf("Senders do not match up! Got %x, expected %x", tx.Input.Address, byteAddr)
}
if tx.Input.Amount != amt {
return fmt.Errorf("Amt does not match up! Got %d, expected %d", tx.Input.Amount, amt)
}
ret := response.Data.Return
if bytes.Compare(ret, returnCode) != 0 {
return fmt.Errorf("Create did not return correct byte code for new contract. Got %x, expected %x", ret, returnCode)
}
return nil
})
// get the return value from a call
data := []byte{0x1} // just needs to be non empty for this to be a CallTx
waitForEvent(t, con, eid, true, func() {
broadcastTx(t, "JSONRPC", byteAddr, contractAddr, data, byteKey, amt, 1000, 1000)
}, func(eid string, b []byte) error {
// unmarshall and assert somethings
var response struct {
Event string
Data struct {
Tx types.CallTx
Return []byte
Exception string
}
Error string
}
var err error
binary.ReadJSON(&response, b, &err)
if err != nil {
return err
}
if response.Error != "" {
return fmt.Errorf(response.Error)
}
ret := response.Data.Return
if bytes.Compare(ret, returnVal) != 0 {
return fmt.Errorf("Call did not return correctly. Got %x, expected %x", ret, returnVal)
}
return nil
})
}

+ 34
- 3
rpc/test/helpers.go View File

@ -6,6 +6,7 @@ import (
"github.com/tendermint/tendermint/account" "github.com/tendermint/tendermint/account"
. "github.com/tendermint/tendermint/common" . "github.com/tendermint/tendermint/common"
"github.com/tendermint/tendermint/config" "github.com/tendermint/tendermint/config"
"github.com/tendermint/tendermint/consensus"
"github.com/tendermint/tendermint/logger" "github.com/tendermint/tendermint/logger"
nm "github.com/tendermint/tendermint/node" nm "github.com/tendermint/tendermint/node"
"github.com/tendermint/tendermint/p2p" "github.com/tendermint/tendermint/p2p"
@ -14,6 +15,7 @@ import (
"github.com/tendermint/tendermint/state" "github.com/tendermint/tendermint/state"
"github.com/tendermint/tendermint/types" "github.com/tendermint/tendermint/types"
"testing" "testing"
"time"
) )
// global variables for use across all tests // global variables for use across all tests
@ -26,9 +28,10 @@ var (
mempoolCount = 0 mempoolCount = 0
userAddr = "D7DFF9806078899C8DA3FE3633CC0BF3C6C2B1BB"
userPriv = "FDE3BD94CB327D19464027BA668194C5EFA46AE83E8419D7542CFF41F00C81972239C21C81EA7173A6C489145490C015E05D4B97448933B708A7EC5B7B4921E3"
userPub = "2239C21C81EA7173A6C489145490C015E05D4B97448933B708A7EC5B7B4921E3"
userAddr = "D7DFF9806078899C8DA3FE3633CC0BF3C6C2B1BB"
userPriv = "FDE3BD94CB327D19464027BA668194C5EFA46AE83E8419D7542CFF41F00C81972239C21C81EA7173A6C489145490C015E05D4B97448933B708A7EC5B7B4921E3"
userPub = "2239C21C81EA7173A6C489145490C015E05D4B97448933B708A7EC5B7B4921E3"
byteAddr, byteKey = initUserBytes()
clients = map[string]cclient.Client{ clients = map[string]cclient.Client{
"JSONRPC": cclient.NewClient(requestAddr, "JSONRPC"), "JSONRPC": cclient.NewClient(requestAddr, "JSONRPC"),
@ -36,6 +39,14 @@ var (
} }
) )
func initUserBytes() ([]byte, [64]byte) {
byteAddr, _ := hex.DecodeString(userAddr)
var byteKey [64]byte
oh, _ := hex.DecodeString(userPriv)
copy(byteKey[:], oh)
return byteAddr, byteKey
}
func decodeHex(hexStr string) []byte { func decodeHex(hexStr string) []byte {
bytes, err := hex.DecodeString(hexStr) bytes, err := hex.DecodeString(hexStr)
if err != nil { if err != nil {
@ -84,6 +95,9 @@ func init() {
priv.SetFile(rootDir + "/priv_validator.json") priv.SetFile(rootDir + "/priv_validator.json")
priv.Save() priv.Save()
consensus.RoundDuration0 = 3 * time.Second
consensus.RoundDurationDelta = 1 * time.Second
// start a node // start a node
ready := make(chan struct{}) ready := make(chan struct{})
go newNode(ready) go newNode(ready)
@ -264,3 +278,20 @@ func checkTx(t *testing.T, fromAddr []byte, priv *account.PrivAccount, tx *types
t.Fatal(types.ErrTxInvalidSignature) t.Fatal(types.ErrTxInvalidSignature)
} }
} }
// simple contract returns 5 + 6 = 0xb
func simpleCallContract() ([]byte, []byte, []byte) {
// this is the code we want to run when the contract is called
contractCode := []byte{0x60, 0x5, 0x60, 0x6, 0x1, 0x60, 0x0, 0x52, 0x60, 0x20, 0x60, 0x0, 0xf3}
// the is the code we need to return the contractCode when the contract is initialized
lenCode := len(contractCode)
// push code to the stack
//code := append([]byte{byte(0x60 + lenCode - 1)}, LeftPadWord256(contractCode).Bytes()...)
code := append([]byte{0x7f}, RightPadWord256(contractCode).Bytes()...)
// store it in memory
code = append(code, []byte{0x60, 0x0, 0x52}...)
// return whats in memory
//code = append(code, []byte{0x60, byte(32 - lenCode), 0x60, byte(lenCode), 0xf3}...)
code = append(code, []byte{0x60, byte(lenCode), 0x60, 0x0, 0xf3}...)
return code, contractCode, LeftPadBytes([]byte{0xb}, 32)
}

+ 1
- 37
rpc/test/tests.go View File

@ -49,11 +49,6 @@ func testGetAccount(t *testing.T, typ string) {
} }
func testSignedTx(t *testing.T, typ string) { func testSignedTx(t *testing.T, typ string) {
byteAddr, _ := hex.DecodeString(userAddr)
var byteKey [64]byte
oh, _ := hex.DecodeString(userPriv)
copy(byteKey[:], oh)
amt := uint64(100) amt := uint64(100)
toAddr := []byte{20, 143, 25, 63, 16, 177, 83, 29, 91, 91, 54, 23, 233, 46, 190, 121, 122, 34, 86, 54} toAddr := []byte{20, 143, 25, 63, 16, 177, 83, 29, 91, 91, 54, 23, 233, 46, 190, 121, 122, 34, 86, 54}
tx, priv := signTx(t, typ, byteAddr, toAddr, nil, byteKey, amt, 0, 0) tx, priv := signTx(t, typ, byteAddr, toAddr, nil, byteKey, amt, 0, 0)
@ -69,11 +64,6 @@ func testSignedTx(t *testing.T, typ string) {
} }
func testBroadcastTx(t *testing.T, typ string) { func testBroadcastTx(t *testing.T, typ string) {
byteAddr, _ := hex.DecodeString(userAddr)
var byteKey [64]byte
oh, _ := hex.DecodeString(userPriv)
copy(byteKey[:], oh)
amt := uint64(100) amt := uint64(100)
toAddr := []byte{20, 143, 25, 63, 16, 177, 83, 29, 91, 91, 54, 23, 233, 46, 190, 121, 122, 34, 86, 54} toAddr := []byte{20, 143, 25, 63, 16, 177, 83, 29, 91, 91, 54, 23, 233, 46, 190, 121, 122, 34, 86, 54}
tx, receipt := broadcastTx(t, typ, byteAddr, toAddr, nil, byteKey, amt, 0, 0) tx, receipt := broadcastTx(t, typ, byteAddr, toAddr, nil, byteKey, amt, 0, 0)
@ -103,11 +93,6 @@ func testGetStorage(t *testing.T, typ string) {
_ = priv _ = priv
//core.SetPrivValidator(priv) //core.SetPrivValidator(priv)
byteAddr, _ := hex.DecodeString(userAddr)
var byteKey [64]byte
oh, _ := hex.DecodeString(userPriv)
copy(byteKey[:], oh)
amt := uint64(1100) amt := uint64(1100)
code := []byte{0x60, 0x5, 0x60, 0x1, 0x55} code := []byte{0x60, 0x5, 0x60, 0x1, 0x55}
_, receipt := broadcastTx(t, typ, byteAddr, nil, code, byteKey, amt, 1000, 1000) _, receipt := broadcastTx(t, typ, byteAddr, nil, code, byteKey, amt, 1000, 1000)
@ -153,30 +138,9 @@ func testCallCode(t *testing.T, typ string) {
func testCall(t *testing.T, typ string) { func testCall(t *testing.T, typ string) {
client := clients[typ] client := clients[typ]
priv := state.LoadPrivValidator(".tendermint/priv_validator.json")
_ = priv
//core.SetPrivValidator(priv)
byteAddr, _ := hex.DecodeString(userAddr)
var byteKey [64]byte
oh, _ := hex.DecodeString(userPriv)
copy(byteKey[:], oh)
// create the contract // create the contract
amt := uint64(6969) amt := uint64(6969)
// this is the code we want to run when the contract is called
contractCode := []byte{0x60, 0x5, 0x60, 0x6, 0x1, 0x60, 0x0, 0x52, 0x60, 0x20, 0x60, 0x0, 0xf3}
// the is the code we need to return the contractCode when the contract is initialized
lenCode := len(contractCode)
// push code to the stack
//code := append([]byte{byte(0x60 + lenCode - 1)}, LeftPadWord256(contractCode).Bytes()...)
code := append([]byte{0x7f}, RightPadWord256(contractCode).Bytes()...)
// store it in memory
code = append(code, []byte{0x60, 0x0, 0x52}...)
// return whats in memory
//code = append(code, []byte{0x60, byte(32 - lenCode), 0x60, byte(lenCode), 0xf3}...)
code = append(code, []byte{0x60, byte(lenCode), 0x60, 0x0, 0xf3}...)
code, _, _ := simpleCallContract()
_, receipt := broadcastTx(t, typ, byteAddr, nil, code, byteKey, amt, 1000, 1000) _, receipt := broadcastTx(t, typ, byteAddr, nil, code, byteKey, amt, 1000, 1000)
if receipt.CreatesContract == 0 { if receipt.CreatesContract == 0 {
t.Fatal("This tx creates a contract") t.Fatal("This tx creates a contract")


+ 24
- 22
state/execution.go View File

@ -13,7 +13,7 @@ import (
// NOTE: If an error occurs during block execution, state will be left // NOTE: If an error occurs during block execution, state will be left
// at an invalid state. Copy the state before calling ExecBlock! // at an invalid state. Copy the state before calling ExecBlock!
func ExecBlock(s *State, block *types.Block, blockPartsHeader types.PartSetHeader) error { func ExecBlock(s *State, block *types.Block, blockPartsHeader types.PartSetHeader) error {
err := execBlock(s, block, blockPartsHeader)
err := execBlock(s, block, blockPartsHeader, true)
if err != nil { if err != nil {
return err return err
} }
@ -29,7 +29,7 @@ func ExecBlock(s *State, block *types.Block, blockPartsHeader types.PartSetHeade
// executes transactions of a block, does not check block.StateHash // executes transactions of a block, does not check block.StateHash
// NOTE: If an error occurs during block execution, state will be left // NOTE: If an error occurs during block execution, state will be left
// at an invalid state. Copy the state before calling execBlock! // at an invalid state. Copy the state before calling execBlock!
func execBlock(s *State, block *types.Block, blockPartsHeader types.PartSetHeader) error {
func execBlock(s *State, block *types.Block, blockPartsHeader types.PartSetHeader, fireEvents bool) error {
// Basic block validation. // Basic block validation.
err := block.ValidateBasic(s.LastBlockHeight, s.LastBlockHash, s.LastBlockParts, s.LastBlockTime) err := block.ValidateBasic(s.LastBlockHeight, s.LastBlockHash, s.LastBlockParts, s.LastBlockTime)
if err != nil { if err != nil {
@ -111,7 +111,7 @@ func execBlock(s *State, block *types.Block, blockPartsHeader types.PartSetHeade
// Commit each tx // Commit each tx
for _, tx := range block.Data.Txs { for _, tx := range block.Data.Txs {
err := ExecTx(blockCache, tx, true)
err := ExecTx(blockCache, tx, true, fireEvents)
if err != nil { if err != nil {
return InvalidTxError{tx, err} return InvalidTxError{tx, err}
} }
@ -291,7 +291,7 @@ func adjustByOutputs(accounts map[string]*account.Account, outs []*types.TxOutpu
// If the tx is invalid, an error will be returned. // If the tx is invalid, an error will be returned.
// Unlike ExecBlock(), state will not be altered. // Unlike ExecBlock(), state will not be altered.
func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall, fireEvents bool) error {
// TODO: do something with fees // TODO: do something with fees
fees := uint64(0) fees := uint64(0)
@ -329,7 +329,7 @@ func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
// If we're in a block (not mempool), // If we're in a block (not mempool),
// fire event on all inputs and outputs // fire event on all inputs and outputs
// see types/events.go for spec // see types/events.go for spec
if runCall {
if fireEvents {
for _, i := range tx.Inputs { for _, i := range tx.Inputs {
_s.evsw.FireEvent(types.EventStringAccInput(i.Address), tx) _s.evsw.FireEvent(types.EventStringAccInput(i.Address), tx)
} }
@ -449,19 +449,21 @@ func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
// Create a receipt from the ret and whether errored. // Create a receipt from the ret and whether errored.
log.Info("VM call complete", "caller", caller, "callee", callee, "return", ret, "err", err) log.Info("VM call complete", "caller", caller, "callee", callee, "return", ret, "err", err)
// Fire Events for sender and receiver
// a separate event will be fired from vm for each
_s.evsw.FireEvent(types.EventStringAccInput(tx.Input.Address), struct {
Tx types.Tx
Return []byte
Exception string
}{tx, ret, exception})
_s.evsw.FireEvent(types.EventStringAccReceive(tx.Address), struct {
Tx types.Tx
Return []byte
Exception string
}{tx, ret, exception})
if fireEvents {
// Fire Events for sender and receiver
// a separate event will be fired from vm for each
_s.evsw.FireEvent(types.EventStringAccInput(tx.Input.Address), struct {
Tx types.Tx
Return []byte
Exception string
}{tx, ret, exception})
_s.evsw.FireEvent(types.EventStringAccReceive(tx.Address), struct {
Tx types.Tx
Return []byte
Exception string
}{tx, ret, exception})
}
} else { } else {
// The mempool does not call txs until // The mempool does not call txs until
// the proposer determines the order of txs. // the proposer determines the order of txs.
@ -529,7 +531,7 @@ func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
if !added { if !added {
panic("Failed to add validator") panic("Failed to add validator")
} }
if runCall {
if fireEvents {
_s.evsw.FireEvent(types.EventStringBond(), tx) _s.evsw.FireEvent(types.EventStringBond(), tx)
} }
return nil return nil
@ -554,7 +556,7 @@ func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
// Good! // Good!
_s.unbondValidator(val) _s.unbondValidator(val)
if runCall {
if fireEvents {
_s.evsw.FireEvent(types.EventStringUnbond(), tx) _s.evsw.FireEvent(types.EventStringUnbond(), tx)
} }
return nil return nil
@ -579,7 +581,7 @@ func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
// Good! // Good!
_s.rebondValidator(val) _s.rebondValidator(val)
if runCall {
if fireEvents {
_s.evsw.FireEvent(types.EventStringRebond(), tx) _s.evsw.FireEvent(types.EventStringRebond(), tx)
} }
return nil return nil
@ -625,7 +627,7 @@ func ExecTx(blockCache *BlockCache, tx_ types.Tx, runCall bool) error {
// Good! (Bad validator!) // Good! (Bad validator!)
_s.destroyValidator(accused) _s.destroyValidator(accused)
if runCall {
if fireEvents {
_s.evsw.FireEvent(types.EventStringDupeout(), tx) _s.evsw.FireEvent(types.EventStringDupeout(), tx)
} }
return nil return nil


+ 1
- 1
state/state.go View File

@ -119,7 +119,7 @@ func (s *State) Hash() []byte {
// Mutates the block in place and updates it with new state hash. // Mutates the block in place and updates it with new state hash.
func (s *State) SetBlockStateHash(block *types.Block) error { func (s *State) SetBlockStateHash(block *types.Block) error {
sCopy := s.Copy() sCopy := s.Copy()
err := execBlock(sCopy, block, types.PartSetHeader{})
err := execBlock(sCopy, block, types.PartSetHeader{}, false) // don't fire events
if err != nil { if err != nil {
return err return err
} }


+ 8
- 0
vm/vm.go View File

@ -6,6 +6,7 @@ import (
"math/big" "math/big"
. "github.com/tendermint/tendermint/common" . "github.com/tendermint/tendermint/common"
"github.com/tendermint/tendermint/events"
"github.com/tendermint/tendermint/vm/sha3" "github.com/tendermint/tendermint/vm/sha3"
) )
@ -45,6 +46,8 @@ type VM struct {
origin Word256 origin Word256
callDepth int callDepth int
evsw *events.EventSwitch
} }
func NewVM(appState AppState, params Params, origin Word256) *VM { func NewVM(appState AppState, params Params, origin Word256) *VM {
@ -56,6 +59,11 @@ func NewVM(appState AppState, params Params, origin Word256) *VM {
} }
} }
// satisfies events.Eventable
func (vm *VM) SetEventSwitch(evsw *events.EventSwitch) {
vm.evsw = evsw
}
// CONTRACT appState is aware of caller and callee, so we can just mutate them. // CONTRACT appState is aware of caller and callee, so we can just mutate them.
// value: To be transferred from caller to callee. Refunded upon error. // value: To be transferred from caller to callee. Refunded upon error.
// gas: Available gas. No refunds for gas. // gas: Available gas. No refunds for gas.


Loading…
Cancel
Save