tendermint/mempool/reactor.go

242 lines
6.3 KiB
Go
Raw Normal View History

2014-09-14 15:37:32 -07:00
package mempool
import (
"bytes"
"errors"
2014-09-14 15:37:32 -07:00
"fmt"
"reflect"
2015-09-25 09:55:59 -07:00
"time"
2014-09-14 15:37:32 -07:00
. "github.com/tendermint/tendermint/common"
"github.com/tendermint/tendermint/events"
2015-04-01 17:30:16 -07:00
"github.com/tendermint/tendermint/p2p"
sm "github.com/tendermint/tendermint/state"
2015-04-01 17:30:16 -07:00
"github.com/tendermint/tendermint/types"
"github.com/tendermint/tendermint/wire"
2014-09-14 15:37:32 -07:00
)
var (
MempoolChannel = byte(0x30)
2015-09-25 09:55:59 -07:00
checkExecutedTxsMilliseconds = 1 // check for new mempool txs to send to peer
txsToSendPerCheck = 64 // send up to this many txs from the mempool per check
newBlockChCapacity = 100 // queue to process this many ResetInfos per peer
2014-09-14 15:37:32 -07:00
)
// MempoolReactor handles mempool tx broadcasting amongst peers.
type MempoolReactor struct {
p2p.BaseReactor
2014-09-14 15:37:32 -07:00
Mempool *Mempool
2015-04-15 23:40:27 -07:00
evsw events.Fireable
2014-09-14 15:37:32 -07:00
}
2014-10-22 17:20:44 -07:00
func NewMempoolReactor(mempool *Mempool) *MempoolReactor {
2014-09-14 15:37:32 -07:00
memR := &MempoolReactor{
Mempool: mempool,
2014-09-14 15:37:32 -07:00
}
memR.BaseReactor = *p2p.NewBaseReactor(log, "MempoolReactor", memR)
2014-09-14 15:37:32 -07:00
return memR
}
2014-10-22 17:20:44 -07:00
// Implements Reactor
func (memR *MempoolReactor) GetChannels() []*p2p.ChannelDescriptor {
return []*p2p.ChannelDescriptor{
&p2p.ChannelDescriptor{
ID: MempoolChannel,
2014-10-22 17:20:44 -07:00
Priority: 5,
},
2014-09-14 15:37:32 -07:00
}
}
// Implements Reactor
2015-09-25 09:55:59 -07:00
func (memR *MempoolReactor) AddPeer(peer *p2p.Peer) {
// Each peer gets a go routine on which we broadcast transactions in the same order we applied them to our state.
newBlockChan := make(chan ResetInfo, newBlockChCapacity)
peer.Data.Set(types.PeerMempoolChKey, newBlockChan)
2015-09-29 08:36:52 -07:00
timer := time.NewTicker(time.Millisecond * time.Duration(checkExecutedTxsMilliseconds))
go memR.broadcastTxRoutine(timer.C, newBlockChan, peer)
2014-09-14 15:37:32 -07:00
}
// Implements Reactor
2015-09-25 09:55:59 -07:00
func (memR *MempoolReactor) RemovePeer(peer *p2p.Peer, reason interface{}) {
// broadcast routine checks if peer is gone and returns
2014-09-14 15:37:32 -07:00
}
2014-10-22 17:20:44 -07:00
// Implements Reactor
func (memR *MempoolReactor) Receive(chID byte, src *p2p.Peer, msgBytes []byte) {
2015-07-13 16:00:01 -07:00
_, msg, err := DecodeMessage(msgBytes)
if err != nil {
2014-12-29 18:39:19 -08:00
log.Warn("Error decoding message", "error", err)
return
}
2015-07-19 14:49:13 -07:00
log.Notice("MempoolReactor received message", "msg", msg)
2014-09-14 15:37:32 -07:00
2015-07-13 16:00:01 -07:00
switch msg := msg.(type) {
2014-09-14 15:37:32 -07:00
case *TxMessage:
err := memR.Mempool.AddTx(msg.Tx)
2014-09-14 15:37:32 -07:00
if err != nil {
// Bad, seen, or conflicting tx.
2015-07-19 14:49:13 -07:00
log.Info("Could not add tx", "tx", msg.Tx)
2014-09-14 15:37:32 -07:00
return
} else {
2015-07-19 14:49:13 -07:00
log.Info("Added valid tx", "tx", msg.Tx)
2014-09-14 15:37:32 -07:00
}
2015-09-25 09:55:59 -07:00
// broadcasting happens from go routines per peer
default:
log.Warn(Fmt("Unknown message type %v", reflect.TypeOf(msg)))
}
}
// "block" is the new block being committed.
// "state" is the result of state.AppendBlock("block").
// Txs that are present in "block" are discarded from mempool.
// Txs that have become invalid in the new "state" are also discarded.
func (memR *MempoolReactor) ResetForBlockAndState(block *types.Block, state *sm.State) {
ri := memR.Mempool.ResetForBlockAndState(block, state)
for _, peer := range memR.Switch.Peers().List() {
peerMempoolCh := peer.Data.Get(types.PeerMempoolChKey).(chan ResetInfo)
select {
case peerMempoolCh <- ri:
default:
memR.Switch.StopPeerForError(peer, errors.New("Peer's mempool push channel full"))
}
}
}
2015-09-25 09:55:59 -07:00
// Just an alias for AddTx since broadcasting happens in peer routines
func (memR *MempoolReactor) BroadcastTx(tx types.Tx) error {
return memR.Mempool.AddTx(tx)
}
type PeerState interface {
GetHeight() int
}
2015-09-29 08:36:52 -07:00
type Peer interface {
IsRunning() bool
Send(byte, interface{}) bool
Get(string) interface{}
}
2015-09-25 09:55:59 -07:00
// send new mempool txs to peer, strictly in order we applied them to our state.
// new blocks take chunks out of the mempool, but we've already sent some txs to the peer.
// so we wait to hear that the peer has progressed to the new height, and then continue sending txs from where we left off
2015-09-29 08:36:52 -07:00
func (memR *MempoolReactor) broadcastTxRoutine(tickerChan <-chan time.Time, newBlockChan chan ResetInfo, peer Peer) {
2015-10-11 14:57:20 -07:00
var height = memR.Mempool.GetHeight()
var txsSent int // new txs sent for height. (reset every new height)
2015-09-25 09:55:59 -07:00
for {
select {
2015-09-29 08:36:52 -07:00
case <-tickerChan:
2015-09-25 09:55:59 -07:00
if !peer.IsRunning() {
return
}
// make sure the peer is up to date
2015-09-29 08:36:52 -07:00
peerState := peer.Get(types.PeerStateKey).(PeerState)
2015-10-11 14:57:20 -07:00
if peerState.GetHeight() < height {
2014-09-14 15:37:32 -07:00
continue
}
2015-09-25 09:55:59 -07:00
// check the mempool for new transactions
2015-10-11 14:57:20 -07:00
newTxs := memR.getNewTxs(height)
txsSentLoop := 0
2015-09-25 09:55:59 -07:00
start := time.Now()
2015-10-11 14:57:20 -07:00
2015-09-25 09:55:59 -07:00
TX_LOOP:
2015-10-11 14:57:20 -07:00
for i := txsSent; i < len(newTxs) && txsSentLoop < txsToSendPerCheck; i++ {
tx := newTxs[i]
2015-09-25 09:55:59 -07:00
msg := &TxMessage{Tx: tx}
success := peer.Send(MempoolChannel, msg)
if !success {
break TX_LOOP
} else {
2015-10-11 14:57:20 -07:00
txsSentLoop += 1
2015-09-25 09:55:59 -07:00
}
}
2015-10-11 14:57:20 -07:00
if txsSentLoop > 0 {
txsSent += txsSentLoop
log.Info("Sent txs to peer", "txsSentLoop", txsSentLoop,
"took", time.Since(start), "txsSent", txsSent, "newTxs", len(newTxs))
2015-09-25 09:55:59 -07:00
}
case ri := <-newBlockChan:
2015-10-11 14:57:20 -07:00
height = ri.Height
2015-09-25 09:55:59 -07:00
// find out how many txs below what we've sent were included in a block and how many became invalid
included := tallyRangesUpTo(ri.Included, txsSent)
invalidated := tallyRangesUpTo(ri.Invalid, txsSent)
txsSent -= included + invalidated
2014-09-14 15:37:32 -07:00
}
2015-09-25 09:55:59 -07:00
}
}
2014-09-14 15:37:32 -07:00
2015-09-25 09:55:59 -07:00
// fetch new txs from the mempool
2015-10-11 14:57:20 -07:00
func (memR *MempoolReactor) getNewTxs(height int) (txs []types.Tx) {
2015-09-25 09:55:59 -07:00
memR.Mempool.mtx.Lock()
defer memR.Mempool.mtx.Unlock()
// if the mempool got ahead of us just return empty txs
if memR.Mempool.state.LastBlockHeight != height {
return
2014-09-14 15:37:32 -07:00
}
2015-10-11 14:57:20 -07:00
return memR.Mempool.txs
2014-09-14 15:37:32 -07:00
}
2015-09-25 09:55:59 -07:00
// return the size of ranges less than upTo
func tallyRangesUpTo(ranger []Range, upTo int) int {
totalUpTo := 0
for _, r := range ranger {
if r.Start >= upTo {
break
}
2015-09-29 08:36:52 -07:00
if r.Start+r.Length >= upTo {
totalUpTo += upTo - r.Start
2015-09-25 09:55:59 -07:00
break
}
totalUpTo += r.Length
2014-10-22 17:20:44 -07:00
}
2015-09-25 09:55:59 -07:00
return totalUpTo
2014-10-22 17:20:44 -07:00
}
// implements events.Eventable
2015-04-15 23:40:27 -07:00
func (memR *MempoolReactor) SetFireable(evsw events.Fireable) {
memR.evsw = evsw
}
2014-09-14 15:37:32 -07:00
//-----------------------------------------------------------------------------
// Messages
const (
2015-04-14 15:57:16 -07:00
msgTypeTx = byte(0x01)
2014-09-14 15:37:32 -07:00
)
2015-04-14 15:57:16 -07:00
type MempoolMessage interface{}
2015-07-25 15:45:45 -07:00
var _ = wire.RegisterInterface(
2015-04-14 15:57:16 -07:00
struct{ MempoolMessage }{},
2015-07-25 15:45:45 -07:00
wire.ConcreteType{&TxMessage{}, msgTypeTx},
2015-04-14 15:57:16 -07:00
)
func DecodeMessage(bz []byte) (msgType byte, msg MempoolMessage, err error) {
2014-09-14 15:37:32 -07:00
msgType = bz[0]
2015-04-14 15:57:16 -07:00
n := new(int64)
r := bytes.NewReader(bz)
2015-07-25 15:45:45 -07:00
msg = wire.ReadBinary(struct{ MempoolMessage }{}, r, n, &err).(struct{ MempoolMessage }).MempoolMessage
2014-09-14 15:37:32 -07:00
return
}
//-------------------------------------
type TxMessage struct {
Tx types.Tx
2014-09-14 15:37:32 -07:00
}
func (m *TxMessage) String() string {
return fmt.Sprintf("[TxMessage %v]", m.Tx)
}