2014-09-11 01:11:41 -07:00
|
|
|
package mempool
|
|
|
|
|
|
|
|
import (
|
2016-02-14 17:00:33 -08:00
|
|
|
"bytes"
|
2015-12-09 14:10:31 -08:00
|
|
|
"container/list"
|
2018-08-28 21:10:06 -07:00
|
|
|
"crypto/sha256"
|
2018-05-11 12:09:16 +04:00
|
|
|
"fmt"
|
2014-09-11 01:11:41 -07:00
|
|
|
"sync"
|
2015-12-01 20:12:01 -08:00
|
|
|
"sync/atomic"
|
2016-02-14 17:00:33 -08:00
|
|
|
"time"
|
2014-09-11 01:11:41 -07:00
|
|
|
|
2017-05-02 11:53:32 +04:00
|
|
|
"github.com/pkg/errors"
|
|
|
|
|
2018-09-21 13:00:36 +04:00
|
|
|
amino "github.com/tendermint/go-amino"
|
2018-06-22 06:59:02 +02:00
|
|
|
abci "github.com/tendermint/tendermint/abci/types"
|
2018-09-21 13:00:36 +04:00
|
|
|
cfg "github.com/tendermint/tendermint/config"
|
2018-07-01 22:36:49 -04:00
|
|
|
auto "github.com/tendermint/tendermint/libs/autofile"
|
|
|
|
"github.com/tendermint/tendermint/libs/clist"
|
|
|
|
cmn "github.com/tendermint/tendermint/libs/common"
|
|
|
|
"github.com/tendermint/tendermint/libs/log"
|
2017-04-28 23:59:02 -04:00
|
|
|
"github.com/tendermint/tendermint/proxy"
|
|
|
|
"github.com/tendermint/tendermint/types"
|
2014-09-11 01:11:41 -07:00
|
|
|
)
|
2014-09-10 02:43:16 -07:00
|
|
|
|
2018-09-23 01:14:05 -04:00
|
|
|
// PreCheckFunc is an optional filter executed before CheckTx and rejects
|
|
|
|
// transaction if false is returned. An example would be to ensure that a
|
|
|
|
// transaction doesn't exceeded the block size.
|
2018-11-05 22:32:52 -08:00
|
|
|
type PreCheckFunc func(types.Tx) error
|
2018-09-21 17:50:06 -07:00
|
|
|
|
|
|
|
// PostCheckFunc is an optional filter executed after CheckTx and rejects
|
|
|
|
// transaction if false is returned. An example would be to ensure a
|
2018-09-23 01:14:05 -04:00
|
|
|
// transaction doesn't require more gas than available for the block.
|
2018-11-05 22:32:52 -08:00
|
|
|
type PostCheckFunc func(types.Tx, *abci.ResponseCheckTx) error
|
2018-09-21 17:50:06 -07:00
|
|
|
|
2015-12-01 20:12:01 -08:00
|
|
|
/*
|
|
|
|
|
2016-01-06 17:14:20 -08:00
|
|
|
The mempool pushes new txs onto the proxyAppConn.
|
2015-12-01 20:12:01 -08:00
|
|
|
It gets a stream of (req, res) tuples from the proxy.
|
2017-10-04 13:14:00 +04:00
|
|
|
The mempool stores good txs in a concurrent linked-list.
|
2015-12-01 20:12:01 -08:00
|
|
|
|
|
|
|
Multiple concurrent go-routines can traverse this linked-list
|
|
|
|
safely by calling .NextWait() on each element.
|
|
|
|
|
|
|
|
So we have several go-routines:
|
|
|
|
1. Consensus calling Update() and Reap() synchronously
|
2016-01-06 17:14:20 -08:00
|
|
|
2. Many mempool reactor's peer routines calling CheckTx()
|
2015-12-01 20:12:01 -08:00
|
|
|
3. Many mempool reactor's peer routines traversing the txs linked list
|
|
|
|
4. Another goroutine calling GarbageCollectTxs() periodically
|
|
|
|
|
|
|
|
To manage these goroutines, there are three methods of locking.
|
|
|
|
1. Mutations to the linked-list is protected by an internal mtx (CList is goroutine-safe)
|
|
|
|
2. Mutations to the linked-list elements are atomic
|
2016-01-06 17:14:20 -08:00
|
|
|
3. CheckTx() calls can be paused upon Update() and Reap(), protected by .proxyMtx
|
2015-12-01 20:12:01 -08:00
|
|
|
|
|
|
|
Garbage collection of old elements from mempool.txs is handlde via
|
|
|
|
the DetachPrev() call, which makes old elements not reachable by
|
|
|
|
peer broadcastTxRoutine() automatically garbage collected.
|
|
|
|
|
2017-01-12 15:53:32 -05:00
|
|
|
TODO: Better handle abci client errors. (make it automatically handle connection errors)
|
2016-02-14 17:00:33 -08:00
|
|
|
|
2015-12-01 20:12:01 -08:00
|
|
|
*/
|
|
|
|
|
2018-05-17 13:38:40 +04:00
|
|
|
var (
|
|
|
|
// ErrTxInCache is returned to the client if we saw tx earlier
|
|
|
|
ErrTxInCache = errors.New("Tx already exists in cache")
|
|
|
|
|
|
|
|
// ErrMempoolIsFull means Tendermint & an application can't handle that much load
|
|
|
|
ErrMempoolIsFull = errors.New("Mempool is full")
|
|
|
|
)
|
2015-12-09 14:10:31 -08:00
|
|
|
|
2018-11-05 22:32:52 -08:00
|
|
|
// ErrPreCheck is returned when tx is too big
|
|
|
|
type ErrPreCheck struct {
|
|
|
|
Reason error
|
|
|
|
}
|
|
|
|
|
|
|
|
func (e ErrPreCheck) Error() string {
|
|
|
|
return e.Reason.Error()
|
|
|
|
}
|
|
|
|
|
|
|
|
// IsPreCheckError returns true if err is due to pre check failure.
|
|
|
|
func IsPreCheckError(err error) bool {
|
|
|
|
_, ok := err.(ErrPreCheck)
|
|
|
|
return ok
|
|
|
|
}
|
|
|
|
|
2018-09-21 17:50:06 -07:00
|
|
|
// PreCheckAminoMaxBytes checks that the size of the transaction plus the amino
|
|
|
|
// overhead is smaller or equal to the expected maxBytes.
|
|
|
|
func PreCheckAminoMaxBytes(maxBytes int64) PreCheckFunc {
|
2018-11-05 22:32:52 -08:00
|
|
|
return func(tx types.Tx) error {
|
2018-09-21 17:50:06 -07:00
|
|
|
// We have to account for the amino overhead in the tx size as well
|
|
|
|
aminoOverhead := amino.UvarintSize(uint64(len(tx)))
|
2018-11-05 22:32:52 -08:00
|
|
|
txSize := int64(len(tx) + aminoOverhead)
|
|
|
|
if txSize > maxBytes {
|
|
|
|
return fmt.Errorf("Tx size (including amino overhead) is too big: %d, max: %d",
|
|
|
|
txSize, maxBytes)
|
|
|
|
}
|
|
|
|
return nil
|
2018-09-21 17:50:06 -07:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
|
|
|
// PostCheckMaxGas checks that the wanted gas is smaller or equal to the passed
|
2018-11-05 22:32:52 -08:00
|
|
|
// maxGas. Returns nil if maxGas is -1.
|
2018-09-21 17:50:06 -07:00
|
|
|
func PostCheckMaxGas(maxGas int64) PostCheckFunc {
|
2018-11-05 22:32:52 -08:00
|
|
|
return func(tx types.Tx, res *abci.ResponseCheckTx) error {
|
2018-09-21 17:50:06 -07:00
|
|
|
if maxGas == -1 {
|
2018-11-05 22:32:52 -08:00
|
|
|
return nil
|
|
|
|
}
|
|
|
|
if res.GasWanted > maxGas {
|
|
|
|
return fmt.Errorf("gas wanted %d is greater than max gas %d",
|
|
|
|
res.GasWanted, maxGas)
|
2018-09-21 17:50:06 -07:00
|
|
|
}
|
2018-11-05 22:32:52 -08:00
|
|
|
return nil
|
2018-09-21 17:50:06 -07:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-06-22 15:06:43 -04:00
|
|
|
// TxID is the hex encoded hash of the bytes as a types.Tx.
|
|
|
|
func TxID(tx []byte) string {
|
|
|
|
return fmt.Sprintf("%X", types.Tx(tx).Hash())
|
|
|
|
}
|
|
|
|
|
2017-10-16 16:01:32 +02:00
|
|
|
// Mempool is an ordered in-memory pool for transactions before they are proposed in a consensus
|
|
|
|
// round. Transaction validity is checked using the CheckTx abci message before the transaction is
|
|
|
|
// added to the pool. The Mempool uses a concurrent list structure for storing transactions that
|
|
|
|
// can be efficiently accessed by multiple concurrent readers.
|
2014-09-10 02:43:16 -07:00
|
|
|
type Mempool struct {
|
2017-05-04 20:07:08 +02:00
|
|
|
config *cfg.MempoolConfig
|
2016-05-08 15:00:58 -07:00
|
|
|
|
2017-07-25 13:57:11 -04:00
|
|
|
proxyMtx sync.Mutex
|
|
|
|
proxyAppConn proxy.AppConnMempool
|
|
|
|
txs *clist.CList // concurrent linked-list of good txs
|
|
|
|
counter int64 // simple incrementing counter
|
2017-12-01 19:04:53 -06:00
|
|
|
height int64 // the last block Update()'d to
|
2017-07-25 13:57:11 -04:00
|
|
|
rechecking int32 // for re-checking filtered txs on Update()
|
|
|
|
recheckCursor *clist.CElement // next expected response
|
|
|
|
recheckEnd *clist.CElement // re-checking stops here
|
2018-06-01 23:59:35 -04:00
|
|
|
notifiedTxsAvailable bool
|
2018-07-23 20:58:24 -04:00
|
|
|
txsAvailable chan struct{} // fires once for each height, when the mempool is not empty
|
2018-09-21 17:50:06 -07:00
|
|
|
preCheck PreCheckFunc
|
|
|
|
postCheck PostCheckFunc
|
2015-12-09 14:10:31 -08:00
|
|
|
|
|
|
|
// Keep a cache of already-seen txs.
|
|
|
|
// This reduces the pressure on the proxyApp.
|
2018-06-19 11:41:16 +04:00
|
|
|
cache txCache
|
2016-10-17 16:54:51 -07:00
|
|
|
|
|
|
|
// A log of mempool txs
|
2016-10-26 21:51:03 -07:00
|
|
|
wal *auto.AutoFile
|
2017-05-02 11:53:32 +04:00
|
|
|
|
|
|
|
logger log.Logger
|
2018-06-15 15:57:11 +04:00
|
|
|
|
|
|
|
metrics *Metrics
|
2014-09-10 02:43:16 -07:00
|
|
|
}
|
|
|
|
|
2018-06-20 10:39:19 +04:00
|
|
|
// MempoolOption sets an optional parameter on the Mempool.
|
|
|
|
type MempoolOption func(*Mempool)
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// NewMempool returns a new Mempool with the given configuration and connection to an application.
|
2018-06-20 10:39:19 +04:00
|
|
|
func NewMempool(
|
|
|
|
config *cfg.MempoolConfig,
|
|
|
|
proxyAppConn proxy.AppConnMempool,
|
|
|
|
height int64,
|
|
|
|
options ...MempoolOption,
|
|
|
|
) *Mempool {
|
2015-12-01 20:12:01 -08:00
|
|
|
mempool := &Mempool{
|
2016-05-08 15:00:58 -07:00
|
|
|
config: config,
|
2016-02-14 17:00:33 -08:00
|
|
|
proxyAppConn: proxyAppConn,
|
|
|
|
txs: clist.New(),
|
|
|
|
counter: 0,
|
2017-07-25 13:57:11 -04:00
|
|
|
height: height,
|
2016-02-14 17:00:33 -08:00
|
|
|
rechecking: 0,
|
|
|
|
recheckCursor: nil,
|
|
|
|
recheckEnd: nil,
|
2017-05-02 11:53:32 +04:00
|
|
|
logger: log.NewNopLogger(),
|
2018-06-16 11:44:03 +04:00
|
|
|
metrics: NopMetrics(),
|
2014-09-10 02:43:16 -07:00
|
|
|
}
|
2018-06-19 11:41:16 +04:00
|
|
|
if config.CacheSize > 0 {
|
|
|
|
mempool.cache = newMapTxCache(config.CacheSize)
|
|
|
|
} else {
|
|
|
|
mempool.cache = nopTxCache{}
|
2014-09-10 02:43:16 -07:00
|
|
|
}
|
2016-01-06 17:14:20 -08:00
|
|
|
proxyAppConn.SetResponseCallback(mempool.resCb)
|
2018-06-16 11:44:03 +04:00
|
|
|
for _, option := range options {
|
|
|
|
option(mempool)
|
|
|
|
}
|
2015-12-01 20:12:01 -08:00
|
|
|
return mempool
|
2014-09-10 02:43:16 -07:00
|
|
|
}
|
|
|
|
|
2017-07-25 13:57:11 -04:00
|
|
|
// EnableTxsAvailable initializes the TxsAvailable channel,
|
2017-07-13 13:19:44 -04:00
|
|
|
// ensuring it will trigger once every height when transactions are available.
|
2017-07-17 13:49:35 -04:00
|
|
|
// NOTE: not thread safe - should only be called once, on startup
|
2017-07-25 13:57:11 -04:00
|
|
|
func (mem *Mempool) EnableTxsAvailable() {
|
2018-07-23 20:58:24 -04:00
|
|
|
mem.txsAvailable = make(chan struct{}, 1)
|
2017-07-13 13:19:44 -04:00
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// SetLogger sets the Logger.
|
2017-05-02 11:53:32 +04:00
|
|
|
func (mem *Mempool) SetLogger(l log.Logger) {
|
|
|
|
mem.logger = l
|
|
|
|
}
|
|
|
|
|
2018-09-21 17:50:06 -07:00
|
|
|
// WithPreCheck sets a filter for the mempool to reject a tx if f(tx) returns
|
|
|
|
// false. This is ran before CheckTx.
|
|
|
|
func WithPreCheck(f PreCheckFunc) MempoolOption {
|
|
|
|
return func(mem *Mempool) { mem.preCheck = f }
|
|
|
|
}
|
|
|
|
|
|
|
|
// WithPostCheck sets a filter for the mempool to reject a tx if f(tx) returns
|
|
|
|
// false. This is ran after CheckTx.
|
|
|
|
func WithPostCheck(f PostCheckFunc) MempoolOption {
|
|
|
|
return func(mem *Mempool) { mem.postCheck = f }
|
2018-08-08 16:03:58 +04:00
|
|
|
}
|
|
|
|
|
2018-06-16 11:44:03 +04:00
|
|
|
// WithMetrics sets the metrics.
|
2018-06-20 10:39:19 +04:00
|
|
|
func WithMetrics(metrics *Metrics) MempoolOption {
|
|
|
|
return func(mem *Mempool) { mem.metrics = metrics }
|
2018-06-16 11:44:03 +04:00
|
|
|
}
|
|
|
|
|
2017-11-23 17:18:05 -07:00
|
|
|
// CloseWAL closes and discards the underlying WAL file.
|
|
|
|
// Any further writes will not be relayed to disk.
|
|
|
|
func (mem *Mempool) CloseWAL() bool {
|
|
|
|
if mem == nil {
|
|
|
|
return false
|
|
|
|
}
|
|
|
|
|
|
|
|
mem.proxyMtx.Lock()
|
|
|
|
defer mem.proxyMtx.Unlock()
|
|
|
|
|
|
|
|
if mem.wal == nil {
|
|
|
|
return false
|
|
|
|
}
|
|
|
|
if err := mem.wal.Close(); err != nil && mem.logger != nil {
|
|
|
|
mem.logger.Error("Mempool.CloseWAL", "err", err)
|
|
|
|
}
|
|
|
|
mem.wal = nil
|
|
|
|
return true
|
|
|
|
}
|
|
|
|
|
2018-01-19 00:57:00 -05:00
|
|
|
func (mem *Mempool) InitWAL() {
|
2017-05-04 21:57:58 +02:00
|
|
|
walDir := mem.config.WalDir()
|
2016-10-30 03:55:27 -07:00
|
|
|
if walDir != "" {
|
2017-04-28 23:59:02 -04:00
|
|
|
err := cmn.EnsureDir(walDir, 0700)
|
2016-11-05 09:15:34 -07:00
|
|
|
if err != nil {
|
2017-05-02 11:53:32 +04:00
|
|
|
cmn.PanicSanity(errors.Wrap(err, "Error ensuring Mempool wal dir"))
|
2016-11-05 09:15:34 -07:00
|
|
|
}
|
2016-10-30 03:55:27 -07:00
|
|
|
af, err := auto.OpenAutoFile(walDir + "/wal")
|
2016-10-17 16:54:51 -07:00
|
|
|
if err != nil {
|
2017-05-02 11:53:32 +04:00
|
|
|
cmn.PanicSanity(errors.Wrap(err, "Error opening Mempool wal file"))
|
2016-10-17 16:54:51 -07:00
|
|
|
}
|
|
|
|
mem.wal = af
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Lock locks the mempool. The consensus must be able to hold lock to safely update.
|
2016-04-11 17:19:56 -04:00
|
|
|
func (mem *Mempool) Lock() {
|
|
|
|
mem.proxyMtx.Lock()
|
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Unlock unlocks the mempool.
|
2016-04-11 17:19:56 -04:00
|
|
|
func (mem *Mempool) Unlock() {
|
|
|
|
mem.proxyMtx.Unlock()
|
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Size returns the number of transactions in the mempool.
|
2016-03-07 18:38:05 -05:00
|
|
|
func (mem *Mempool) Size() int {
|
|
|
|
return mem.txs.Len()
|
|
|
|
}
|
|
|
|
|
2018-01-23 16:56:14 +04:00
|
|
|
// Flushes the mempool connection to ensure async resCb calls are done e.g.
|
|
|
|
// from CheckTx.
|
|
|
|
func (mem *Mempool) FlushAppConn() error {
|
|
|
|
return mem.proxyAppConn.FlushSync()
|
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Flush removes all transactions from the mempool and cache
|
2016-07-05 14:41:50 -04:00
|
|
|
func (mem *Mempool) Flush() {
|
|
|
|
mem.proxyMtx.Lock()
|
|
|
|
defer mem.proxyMtx.Unlock()
|
|
|
|
|
2016-10-07 15:30:17 -07:00
|
|
|
mem.cache.Reset()
|
2016-07-05 14:41:50 -04:00
|
|
|
|
|
|
|
for e := mem.txs.Front(); e != nil; e = e.Next() {
|
|
|
|
mem.txs.Remove(e)
|
|
|
|
e.DetachPrev()
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-02-01 18:33:33 +04:00
|
|
|
// TxsFront returns the first transaction in the ordered list for peer
|
|
|
|
// goroutines to call .NextWait() on.
|
|
|
|
func (mem *Mempool) TxsFront() *clist.CElement {
|
|
|
|
return mem.txs.Front()
|
|
|
|
}
|
|
|
|
|
|
|
|
// TxsWaitChan returns a channel to wait on transactions. It will be closed
|
|
|
|
// once the mempool is not empty (ie. the internal `mem.txs` has at least one
|
|
|
|
// element)
|
|
|
|
func (mem *Mempool) TxsWaitChan() <-chan struct{} {
|
|
|
|
return mem.txs.WaitChan()
|
2015-03-21 13:31:17 -07:00
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// CheckTx executes a new transaction against the application to determine its validity
|
|
|
|
// and whether it should be added to the mempool.
|
|
|
|
// It blocks if we're waiting on Update() or Reap().
|
2016-02-08 00:48:58 -08:00
|
|
|
// cb: A callback from the CheckTx command.
|
|
|
|
// It gets called from another goroutine.
|
2016-02-08 15:22:49 -08:00
|
|
|
// CONTRACT: Either cb will get called, or err returned.
|
2017-01-12 15:53:32 -05:00
|
|
|
func (mem *Mempool) CheckTx(tx types.Tx, cb func(*abci.Response)) (err error) {
|
2015-12-01 20:12:01 -08:00
|
|
|
mem.proxyMtx.Lock()
|
|
|
|
defer mem.proxyMtx.Unlock()
|
2015-09-29 11:36:52 -04:00
|
|
|
|
2018-05-17 13:38:40 +04:00
|
|
|
if mem.Size() >= mem.config.Size {
|
|
|
|
return ErrMempoolIsFull
|
|
|
|
}
|
|
|
|
|
2018-11-05 22:32:52 -08:00
|
|
|
if mem.preCheck != nil {
|
|
|
|
if err := mem.preCheck(tx); err != nil {
|
|
|
|
return ErrPreCheck{err}
|
|
|
|
}
|
2018-08-08 16:03:58 +04:00
|
|
|
}
|
|
|
|
|
2015-12-09 14:10:31 -08:00
|
|
|
// CACHE
|
2018-05-17 11:52:07 +04:00
|
|
|
if !mem.cache.Push(tx) {
|
2018-01-19 00:57:00 -05:00
|
|
|
return ErrTxInCache
|
2015-12-09 14:10:31 -08:00
|
|
|
}
|
|
|
|
// END CACHE
|
|
|
|
|
2016-10-17 16:54:51 -07:00
|
|
|
// WAL
|
|
|
|
if mem.wal != nil {
|
|
|
|
// TODO: Notify administrators when WAL fails
|
2017-09-06 13:11:47 -04:00
|
|
|
_, err := mem.wal.Write([]byte(tx))
|
|
|
|
if err != nil {
|
2017-09-21 10:56:42 -04:00
|
|
|
mem.logger.Error("Error writing to WAL", "err", err)
|
2017-09-06 13:11:47 -04:00
|
|
|
}
|
|
|
|
_, err = mem.wal.Write([]byte("\n"))
|
|
|
|
if err != nil {
|
2017-09-21 10:56:42 -04:00
|
|
|
mem.logger.Error("Error writing to WAL", "err", err)
|
2017-09-06 13:11:47 -04:00
|
|
|
}
|
2016-10-17 16:54:51 -07:00
|
|
|
}
|
|
|
|
// END WAL
|
|
|
|
|
2016-02-14 17:00:33 -08:00
|
|
|
// NOTE: proxyAppConn may error if tx buffer is full
|
2016-01-06 17:14:20 -08:00
|
|
|
if err = mem.proxyAppConn.Error(); err != nil {
|
2014-09-10 16:56:02 -07:00
|
|
|
return err
|
2014-09-10 02:43:16 -07:00
|
|
|
}
|
2016-02-08 00:48:58 -08:00
|
|
|
reqRes := mem.proxyAppConn.CheckTxAsync(tx)
|
|
|
|
if cb != nil {
|
|
|
|
reqRes.SetCallback(cb)
|
|
|
|
}
|
|
|
|
|
2015-12-01 20:12:01 -08:00
|
|
|
return nil
|
2014-09-10 02:43:16 -07:00
|
|
|
}
|
2014-09-11 01:11:41 -07:00
|
|
|
|
2017-01-12 15:53:32 -05:00
|
|
|
// ABCI callback function
|
|
|
|
func (mem *Mempool) resCb(req *abci.Request, res *abci.Response) {
|
2016-02-14 17:00:33 -08:00
|
|
|
if mem.recheckCursor == nil {
|
|
|
|
mem.resCbNormal(req, res)
|
|
|
|
} else {
|
2018-10-10 09:27:43 -07:00
|
|
|
mem.metrics.RecheckTimes.Add(1)
|
2016-02-14 17:00:33 -08:00
|
|
|
mem.resCbRecheck(req, res)
|
|
|
|
}
|
2018-06-15 15:57:11 +04:00
|
|
|
mem.metrics.Size.Set(float64(mem.Size()))
|
2016-02-14 17:00:33 -08:00
|
|
|
}
|
|
|
|
|
2017-01-12 15:53:32 -05:00
|
|
|
func (mem *Mempool) resCbNormal(req *abci.Request, res *abci.Response) {
|
2016-05-14 12:33:27 -04:00
|
|
|
switch r := res.Value.(type) {
|
2017-01-12 15:53:32 -05:00
|
|
|
case *abci.Response_CheckTx:
|
2017-09-05 16:08:12 -04:00
|
|
|
tx := req.GetCheckTx().Tx
|
2018-09-21 17:50:06 -07:00
|
|
|
if (r.CheckTx.Code == abci.CodeTypeOK) &&
|
|
|
|
mem.isPostCheckPass(tx, r.CheckTx) {
|
2016-01-06 17:14:20 -08:00
|
|
|
mem.counter++
|
|
|
|
memTx := &mempoolTx{
|
2018-09-12 13:41:19 -07:00
|
|
|
counter: mem.counter,
|
|
|
|
height: mem.height,
|
|
|
|
gasWanted: r.CheckTx.GasWanted,
|
|
|
|
tx: tx,
|
2015-12-01 20:12:01 -08:00
|
|
|
}
|
2016-01-06 17:14:20 -08:00
|
|
|
mem.txs.PushBack(memTx)
|
2018-11-05 22:32:52 -08:00
|
|
|
mem.logger.Info("Added good transaction",
|
|
|
|
"tx", TxID(tx),
|
|
|
|
"res", r,
|
|
|
|
"height", memTx.height,
|
|
|
|
"total", mem.Size(),
|
|
|
|
"counter", memTx.counter,
|
|
|
|
)
|
2018-10-10 09:27:43 -07:00
|
|
|
mem.metrics.TxSizeBytes.Observe(float64(len(tx)))
|
2017-08-04 21:36:11 -04:00
|
|
|
mem.notifyTxsAvailable()
|
2016-01-06 17:14:20 -08:00
|
|
|
} else {
|
|
|
|
// ignore bad transaction
|
2018-06-22 15:06:43 -04:00
|
|
|
mem.logger.Info("Rejected bad transaction", "tx", TxID(tx), "res", r)
|
2018-10-10 09:27:43 -07:00
|
|
|
mem.metrics.FailedTxs.Add(1)
|
2016-06-23 20:53:11 -04:00
|
|
|
// remove from cache (it might be good later)
|
2017-09-05 16:08:12 -04:00
|
|
|
mem.cache.Remove(tx)
|
2015-12-01 20:12:01 -08:00
|
|
|
}
|
|
|
|
default:
|
|
|
|
// ignore other messages
|
|
|
|
}
|
2014-09-11 01:11:41 -07:00
|
|
|
}
|
|
|
|
|
2017-01-12 15:53:32 -05:00
|
|
|
func (mem *Mempool) resCbRecheck(req *abci.Request, res *abci.Response) {
|
2016-05-14 12:33:27 -04:00
|
|
|
switch r := res.Value.(type) {
|
2017-01-12 15:53:32 -05:00
|
|
|
case *abci.Response_CheckTx:
|
2016-02-14 17:00:33 -08:00
|
|
|
memTx := mem.recheckCursor.Value.(*mempoolTx)
|
2016-05-14 12:33:27 -04:00
|
|
|
if !bytes.Equal(req.GetCheckTx().Tx, memTx.tx) {
|
2018-09-21 17:50:06 -07:00
|
|
|
cmn.PanicSanity(
|
|
|
|
fmt.Sprintf(
|
|
|
|
"Unexpected tx response from proxy during recheck\nExpected %X, got %X",
|
|
|
|
r.CheckTx.Data,
|
|
|
|
memTx.tx,
|
|
|
|
),
|
|
|
|
)
|
2016-02-14 17:00:33 -08:00
|
|
|
}
|
2018-09-21 17:50:06 -07:00
|
|
|
if (r.CheckTx.Code == abci.CodeTypeOK) && mem.isPostCheckPass(memTx.tx, r.CheckTx) {
|
2016-02-14 17:00:33 -08:00
|
|
|
// Good, nothing to do.
|
|
|
|
} else {
|
|
|
|
// Tx became invalidated due to newly committed block.
|
|
|
|
mem.txs.Remove(mem.recheckCursor)
|
|
|
|
mem.recheckCursor.DetachPrev()
|
2016-06-23 20:53:11 -04:00
|
|
|
|
|
|
|
// remove from cache (it might be good later)
|
2016-10-07 15:30:17 -07:00
|
|
|
mem.cache.Remove(req.GetCheckTx().Tx)
|
2016-02-14 17:00:33 -08:00
|
|
|
}
|
|
|
|
if mem.recheckCursor == mem.recheckEnd {
|
|
|
|
mem.recheckCursor = nil
|
|
|
|
} else {
|
|
|
|
mem.recheckCursor = mem.recheckCursor.Next()
|
|
|
|
}
|
|
|
|
if mem.recheckCursor == nil {
|
|
|
|
// Done!
|
|
|
|
atomic.StoreInt32(&mem.rechecking, 0)
|
2017-05-02 11:53:32 +04:00
|
|
|
mem.logger.Info("Done rechecking txs")
|
2017-07-12 01:02:16 -04:00
|
|
|
|
2017-10-18 16:38:52 +04:00
|
|
|
// incase the recheck removed all txs
|
|
|
|
if mem.Size() > 0 {
|
|
|
|
mem.notifyTxsAvailable()
|
|
|
|
}
|
2016-02-14 17:00:33 -08:00
|
|
|
}
|
|
|
|
default:
|
|
|
|
// ignore other messages
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2017-07-13 13:19:44 -04:00
|
|
|
// TxsAvailable returns a channel which fires once for every height,
|
|
|
|
// and only when transactions are available in the mempool.
|
2017-07-25 13:57:11 -04:00
|
|
|
// NOTE: the returned channel may be nil if EnableTxsAvailable was not called.
|
2018-07-23 20:58:24 -04:00
|
|
|
func (mem *Mempool) TxsAvailable() <-chan struct{} {
|
2017-07-12 01:02:16 -04:00
|
|
|
return mem.txsAvailable
|
|
|
|
}
|
|
|
|
|
2017-08-04 21:36:11 -04:00
|
|
|
func (mem *Mempool) notifyTxsAvailable() {
|
2017-07-25 13:57:11 -04:00
|
|
|
if mem.Size() == 0 {
|
|
|
|
panic("notified txs available but mempool is empty!")
|
|
|
|
}
|
2017-10-16 16:01:32 +02:00
|
|
|
if mem.txsAvailable != nil && !mem.notifiedTxsAvailable {
|
2018-06-18 17:08:09 -07:00
|
|
|
// channel cap is 1, so this will send once
|
2018-07-23 13:34:45 +02:00
|
|
|
mem.notifiedTxsAvailable = true
|
2018-06-01 23:59:35 -04:00
|
|
|
select {
|
2018-07-23 20:58:24 -04:00
|
|
|
case mem.txsAvailable <- struct{}{}:
|
2018-06-01 23:59:35 -04:00
|
|
|
default:
|
|
|
|
}
|
2017-07-13 13:19:44 -04:00
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-09-12 13:41:19 -07:00
|
|
|
// ReapMaxBytesMaxGas reaps transactions from the mempool up to maxBytes bytes total
|
|
|
|
// with the condition that the total gasWanted must be less than maxGas.
|
|
|
|
// If both maxes are negative, there is no cap on the size of all returned
|
2018-08-08 16:03:58 +04:00
|
|
|
// transactions (~ all available transactions).
|
2018-09-21 13:00:36 +04:00
|
|
|
func (mem *Mempool) ReapMaxBytesMaxGas(maxBytes, maxGas int64) types.Txs {
|
2015-12-01 20:12:01 -08:00
|
|
|
mem.proxyMtx.Lock()
|
|
|
|
defer mem.proxyMtx.Unlock()
|
|
|
|
|
2016-02-14 17:00:33 -08:00
|
|
|
for atomic.LoadInt32(&mem.rechecking) > 0 {
|
|
|
|
// TODO: Something better?
|
|
|
|
time.Sleep(time.Millisecond * 10)
|
|
|
|
}
|
2015-12-01 20:12:01 -08:00
|
|
|
|
2018-09-21 13:00:36 +04:00
|
|
|
var totalBytes int64
|
2018-09-12 13:41:19 -07:00
|
|
|
var totalGas int64
|
2018-08-08 16:03:58 +04:00
|
|
|
// TODO: we will get a performance boost if we have a good estimate of avg
|
|
|
|
// size per tx, and set the initial capacity based off of that.
|
|
|
|
// txs := make([]types.Tx, 0, cmn.MinInt(mem.txs.Len(), max/mem.avgTxSize))
|
|
|
|
txs := make([]types.Tx, 0, mem.txs.Len())
|
|
|
|
for e := mem.txs.Front(); e != nil; e = e.Next() {
|
|
|
|
memTx := e.Value.(*mempoolTx)
|
2018-09-12 13:41:19 -07:00
|
|
|
// Check total size requirement
|
2018-09-21 13:00:36 +04:00
|
|
|
aminoOverhead := int64(amino.UvarintSize(uint64(len(memTx.tx))))
|
|
|
|
if maxBytes > -1 && totalBytes+int64(len(memTx.tx))+aminoOverhead > maxBytes {
|
2018-09-12 13:41:19 -07:00
|
|
|
return txs
|
|
|
|
}
|
2018-09-21 13:00:36 +04:00
|
|
|
totalBytes += int64(len(memTx.tx)) + aminoOverhead
|
2018-09-12 13:41:19 -07:00
|
|
|
// Check total gas requirement
|
|
|
|
if maxGas > -1 && totalGas+memTx.gasWanted > maxGas {
|
2018-08-08 16:03:58 +04:00
|
|
|
return txs
|
|
|
|
}
|
2018-09-12 13:41:19 -07:00
|
|
|
totalGas += memTx.gasWanted
|
2018-08-08 16:03:58 +04:00
|
|
|
txs = append(txs, memTx.tx)
|
|
|
|
}
|
2016-02-14 17:00:33 -08:00
|
|
|
return txs
|
2015-09-25 12:55:59 -04:00
|
|
|
}
|
|
|
|
|
2018-08-08 16:03:58 +04:00
|
|
|
// ReapMaxTxs reaps up to max transactions from the mempool.
|
max-bytes PR follow-up (#2318)
* ReapMaxTxs: return all txs if max is negative
this mirrors ReapMaxBytes behavior
See https://github.com/tendermint/tendermint/pull/2184#discussion_r214439950
* increase MaxAminoOverheadForBlock
tested with:
```
func TestMaxAminoOverheadForBlock(t *testing.T) {
maxChainID := ""
for i := 0; i < MaxChainIDLen; i++ {
maxChainID += "𠜎"
}
h := Header{
ChainID: maxChainID,
Height: 10,
Time: time.Now().UTC(),
NumTxs: 100,
TotalTxs: 200,
LastBlockID: makeBlockID(make([]byte, 20), 300, make([]byte, 20)),
LastCommitHash: tmhash.Sum([]byte("last_commit_hash")),
DataHash: tmhash.Sum([]byte("data_hash")),
ValidatorsHash: tmhash.Sum([]byte("validators_hash")),
NextValidatorsHash: tmhash.Sum([]byte("next_validators_hash")),
ConsensusHash: tmhash.Sum([]byte("consensus_hash")),
AppHash: tmhash.Sum([]byte("app_hash")),
LastResultsHash: tmhash.Sum([]byte("last_results_hash")),
EvidenceHash: tmhash.Sum([]byte("evidence_hash")),
ProposerAddress: tmhash.Sum([]byte("proposer_address")),
}
b := Block{
Header: h,
Data: Data{Txs: makeTxs(10000, 100)},
Evidence: EvidenceData{},
LastCommit: &Commit{},
}
bz, err := cdc.MarshalBinary(b)
require.NoError(t, err)
assert.Equal(t, MaxHeaderBytes+MaxAminoOverheadForBlock-2, len(bz)-1000000-20000-1)
}
```
* fix MaxYYY constants calculation
by using math.MaxInt64
See https://github.com/tendermint/tendermint/pull/2184#discussion_r214444244
* pass mempool filter as an option
See https://github.com/tendermint/tendermint/pull/2184#discussion_r214445869
* fixes after Dev's comments
2018-09-04 11:46:34 +04:00
|
|
|
// If max is negative, there is no cap on the size of all returned
|
|
|
|
// transactions (~ all available transactions).
|
2018-08-08 16:03:58 +04:00
|
|
|
func (mem *Mempool) ReapMaxTxs(max int) types.Txs {
|
|
|
|
mem.proxyMtx.Lock()
|
|
|
|
defer mem.proxyMtx.Unlock()
|
|
|
|
|
|
|
|
if max < 0 {
|
max-bytes PR follow-up (#2318)
* ReapMaxTxs: return all txs if max is negative
this mirrors ReapMaxBytes behavior
See https://github.com/tendermint/tendermint/pull/2184#discussion_r214439950
* increase MaxAminoOverheadForBlock
tested with:
```
func TestMaxAminoOverheadForBlock(t *testing.T) {
maxChainID := ""
for i := 0; i < MaxChainIDLen; i++ {
maxChainID += "𠜎"
}
h := Header{
ChainID: maxChainID,
Height: 10,
Time: time.Now().UTC(),
NumTxs: 100,
TotalTxs: 200,
LastBlockID: makeBlockID(make([]byte, 20), 300, make([]byte, 20)),
LastCommitHash: tmhash.Sum([]byte("last_commit_hash")),
DataHash: tmhash.Sum([]byte("data_hash")),
ValidatorsHash: tmhash.Sum([]byte("validators_hash")),
NextValidatorsHash: tmhash.Sum([]byte("next_validators_hash")),
ConsensusHash: tmhash.Sum([]byte("consensus_hash")),
AppHash: tmhash.Sum([]byte("app_hash")),
LastResultsHash: tmhash.Sum([]byte("last_results_hash")),
EvidenceHash: tmhash.Sum([]byte("evidence_hash")),
ProposerAddress: tmhash.Sum([]byte("proposer_address")),
}
b := Block{
Header: h,
Data: Data{Txs: makeTxs(10000, 100)},
Evidence: EvidenceData{},
LastCommit: &Commit{},
}
bz, err := cdc.MarshalBinary(b)
require.NoError(t, err)
assert.Equal(t, MaxHeaderBytes+MaxAminoOverheadForBlock-2, len(bz)-1000000-20000-1)
}
```
* fix MaxYYY constants calculation
by using math.MaxInt64
See https://github.com/tendermint/tendermint/pull/2184#discussion_r214444244
* pass mempool filter as an option
See https://github.com/tendermint/tendermint/pull/2184#discussion_r214445869
* fixes after Dev's comments
2018-09-04 11:46:34 +04:00
|
|
|
max = mem.txs.Len()
|
2018-08-08 16:03:58 +04:00
|
|
|
}
|
|
|
|
|
|
|
|
for atomic.LoadInt32(&mem.rechecking) > 0 {
|
|
|
|
// TODO: Something better?
|
|
|
|
time.Sleep(time.Millisecond * 10)
|
2016-03-06 15:08:32 -08:00
|
|
|
}
|
2018-08-08 16:03:58 +04:00
|
|
|
|
|
|
|
txs := make([]types.Tx, 0, cmn.MinInt(mem.txs.Len(), max))
|
|
|
|
for e := mem.txs.Front(); e != nil && len(txs) <= max; e = e.Next() {
|
2015-12-01 20:12:01 -08:00
|
|
|
memTx := e.Value.(*mempoolTx)
|
|
|
|
txs = append(txs, memTx.tx)
|
|
|
|
}
|
|
|
|
return txs
|
2015-09-25 12:55:59 -04:00
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Update informs the mempool that the given txs were committed and can be discarded.
|
2015-12-01 20:12:01 -08:00
|
|
|
// NOTE: this should be called *after* block is committed by consensus.
|
2016-04-11 17:19:56 -04:00
|
|
|
// NOTE: unsafe; Lock/Unlock must be managed by caller
|
2018-09-21 17:50:06 -07:00
|
|
|
func (mem *Mempool) Update(
|
|
|
|
height int64,
|
|
|
|
txs types.Txs,
|
|
|
|
preCheck PreCheckFunc,
|
|
|
|
postCheck PostCheckFunc,
|
|
|
|
) error {
|
2016-01-06 17:14:20 -08:00
|
|
|
// First, create a lookup map of txns in new txs.
|
2018-08-08 01:50:10 +09:00
|
|
|
txsMap := make(map[string]struct{}, len(txs))
|
2016-01-06 17:14:20 -08:00
|
|
|
for _, tx := range txs {
|
|
|
|
txsMap[string(tx)] = struct{}{}
|
2015-12-01 20:12:01 -08:00
|
|
|
}
|
2014-09-11 01:11:41 -07:00
|
|
|
|
2016-01-06 17:14:20 -08:00
|
|
|
// Set height
|
|
|
|
mem.height = height
|
2017-07-12 01:02:16 -04:00
|
|
|
mem.notifiedTxsAvailable = false
|
|
|
|
|
2018-09-21 17:50:06 -07:00
|
|
|
if preCheck != nil {
|
|
|
|
mem.preCheck = preCheck
|
|
|
|
}
|
|
|
|
if postCheck != nil {
|
|
|
|
mem.postCheck = postCheck
|
2018-08-08 16:03:58 +04:00
|
|
|
}
|
|
|
|
|
2016-01-06 17:14:20 -08:00
|
|
|
// Remove transactions that are already in txs.
|
2016-02-14 17:00:33 -08:00
|
|
|
goodTxs := mem.filterTxs(txsMap)
|
2016-04-19 21:18:40 -04:00
|
|
|
// Recheck mempool txs if any txs were committed in the block
|
2018-09-30 13:28:34 -04:00
|
|
|
if mem.config.Recheck && len(goodTxs) > 0 {
|
2017-07-12 01:02:16 -04:00
|
|
|
mem.logger.Info("Recheck txs", "numtxs", len(goodTxs), "height", height)
|
2016-02-29 16:15:23 -05:00
|
|
|
mem.recheckTxs(goodTxs)
|
|
|
|
// At this point, mem.txs are being rechecked.
|
|
|
|
// mem.recheckCursor re-scans mem.txs and possibly removes some txs.
|
|
|
|
// Before mem.Reap(), we should wait for mem.recheckCursor to be nil.
|
|
|
|
}
|
2018-08-08 16:03:58 +04:00
|
|
|
|
|
|
|
// Update metrics
|
2018-06-15 15:57:11 +04:00
|
|
|
mem.metrics.Size.Set(float64(mem.Size()))
|
2018-08-08 16:03:58 +04:00
|
|
|
|
2017-09-21 09:55:06 -04:00
|
|
|
return nil
|
2015-09-25 12:55:59 -04:00
|
|
|
}
|
|
|
|
|
2016-01-06 17:14:20 -08:00
|
|
|
func (mem *Mempool) filterTxs(blockTxsMap map[string]struct{}) []types.Tx {
|
2015-12-01 20:12:01 -08:00
|
|
|
goodTxs := make([]types.Tx, 0, mem.txs.Len())
|
|
|
|
for e := mem.txs.Front(); e != nil; e = e.Next() {
|
|
|
|
memTx := e.Value.(*mempoolTx)
|
2016-07-11 12:32:24 -04:00
|
|
|
// Remove the tx if it's alredy in a block.
|
2015-12-01 20:12:01 -08:00
|
|
|
if _, ok := blockTxsMap[string(memTx.tx)]; ok {
|
2016-06-23 20:53:11 -04:00
|
|
|
// remove from clist
|
2015-12-01 20:12:01 -08:00
|
|
|
mem.txs.Remove(e)
|
|
|
|
e.DetachPrev()
|
2016-06-23 20:53:11 -04:00
|
|
|
|
2016-07-11 12:32:24 -04:00
|
|
|
// NOTE: we don't remove committed txs from the cache.
|
2015-12-01 20:12:01 -08:00
|
|
|
continue
|
|
|
|
}
|
|
|
|
// Good tx!
|
|
|
|
goodTxs = append(goodTxs, memTx.tx)
|
2015-09-25 12:55:59 -04:00
|
|
|
}
|
2015-12-01 20:12:01 -08:00
|
|
|
return goodTxs
|
2015-09-25 12:55:59 -04:00
|
|
|
}
|
|
|
|
|
2016-02-14 17:00:33 -08:00
|
|
|
// NOTE: pass in goodTxs because mem.txs can mutate concurrently.
|
|
|
|
func (mem *Mempool) recheckTxs(goodTxs []types.Tx) {
|
|
|
|
if len(goodTxs) == 0 {
|
|
|
|
return
|
|
|
|
}
|
|
|
|
atomic.StoreInt32(&mem.rechecking, 1)
|
|
|
|
mem.recheckCursor = mem.txs.Front()
|
|
|
|
mem.recheckEnd = mem.txs.Back()
|
|
|
|
|
|
|
|
// Push txs to proxyAppConn
|
|
|
|
// NOTE: resCb() may be called concurrently.
|
|
|
|
for _, tx := range goodTxs {
|
|
|
|
mem.proxyAppConn.CheckTxAsync(tx)
|
|
|
|
}
|
|
|
|
mem.proxyAppConn.FlushAsync()
|
|
|
|
}
|
|
|
|
|
2018-09-21 17:50:06 -07:00
|
|
|
func (mem *Mempool) isPostCheckPass(tx types.Tx, r *abci.ResponseCheckTx) bool {
|
2018-11-05 22:32:52 -08:00
|
|
|
if mem.postCheck == nil {
|
|
|
|
return true
|
|
|
|
}
|
|
|
|
if err := mem.postCheck(tx, r); err != nil {
|
|
|
|
return false
|
|
|
|
}
|
|
|
|
return true
|
2018-09-21 17:50:06 -07:00
|
|
|
}
|
|
|
|
|
2015-12-01 20:12:01 -08:00
|
|
|
//--------------------------------------------------------------------------------
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// mempoolTx is a transaction that successfully ran
|
2015-12-01 20:12:01 -08:00
|
|
|
type mempoolTx struct {
|
2018-09-12 13:41:19 -07:00
|
|
|
counter int64 // a simple incrementing counter
|
|
|
|
height int64 // height that this tx had been validated in
|
|
|
|
gasWanted int64 // amount of gas this tx states it will require
|
|
|
|
tx types.Tx //
|
2015-12-01 20:12:01 -08:00
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Height returns the height for this transaction
|
2017-12-01 19:04:53 -06:00
|
|
|
func (memTx *mempoolTx) Height() int64 {
|
|
|
|
return atomic.LoadInt64(&memTx.height)
|
2014-09-11 01:11:41 -07:00
|
|
|
}
|
2016-10-07 15:30:17 -07:00
|
|
|
|
|
|
|
//--------------------------------------------------------------------------------
|
|
|
|
|
2018-06-19 11:41:16 +04:00
|
|
|
type txCache interface {
|
|
|
|
Reset()
|
|
|
|
Push(tx types.Tx) bool
|
|
|
|
Remove(tx types.Tx)
|
|
|
|
}
|
|
|
|
|
2018-08-28 21:10:06 -07:00
|
|
|
// mapTxCache maintains a cache of transactions. This only stores
|
|
|
|
// the hash of the tx, due to memory concerns.
|
2018-06-19 11:41:16 +04:00
|
|
|
type mapTxCache struct {
|
2016-10-07 15:30:17 -07:00
|
|
|
mtx sync.Mutex
|
|
|
|
size int
|
2018-08-28 21:10:06 -07:00
|
|
|
map_ map[[sha256.Size]byte]*list.Element
|
2016-10-07 15:30:17 -07:00
|
|
|
list *list.List // to remove oldest tx when cache gets too big
|
|
|
|
}
|
|
|
|
|
2018-06-19 11:41:16 +04:00
|
|
|
var _ txCache = (*mapTxCache)(nil)
|
|
|
|
|
|
|
|
// newMapTxCache returns a new mapTxCache.
|
|
|
|
func newMapTxCache(cacheSize int) *mapTxCache {
|
|
|
|
return &mapTxCache{
|
2016-10-07 15:30:17 -07:00
|
|
|
size: cacheSize,
|
2018-08-28 21:10:06 -07:00
|
|
|
map_: make(map[[sha256.Size]byte]*list.Element, cacheSize),
|
2016-10-07 15:30:17 -07:00
|
|
|
list: list.New(),
|
|
|
|
}
|
|
|
|
}
|
|
|
|
|
2018-06-19 11:41:16 +04:00
|
|
|
// Reset resets the cache to an empty state.
|
|
|
|
func (cache *mapTxCache) Reset() {
|
2016-10-07 15:30:17 -07:00
|
|
|
cache.mtx.Lock()
|
2018-08-28 21:10:06 -07:00
|
|
|
cache.map_ = make(map[[sha256.Size]byte]*list.Element, cache.size)
|
2016-10-07 15:30:17 -07:00
|
|
|
cache.list.Init()
|
|
|
|
cache.mtx.Unlock()
|
|
|
|
}
|
|
|
|
|
2018-06-19 11:41:16 +04:00
|
|
|
// Push adds the given tx to the cache and returns true. It returns false if tx
|
|
|
|
// is already in the cache.
|
|
|
|
func (cache *mapTxCache) Push(tx types.Tx) bool {
|
2016-10-07 15:30:17 -07:00
|
|
|
cache.mtx.Lock()
|
|
|
|
defer cache.mtx.Unlock()
|
|
|
|
|
2018-08-28 21:10:06 -07:00
|
|
|
// Use the tx hash in the cache
|
|
|
|
txHash := sha256.Sum256(tx)
|
2018-09-20 01:40:22 +08:00
|
|
|
if moved, exists := cache.map_[txHash]; exists {
|
|
|
|
cache.list.MoveToFront(moved)
|
2016-10-07 15:30:17 -07:00
|
|
|
return false
|
|
|
|
}
|
|
|
|
|
|
|
|
if cache.list.Len() >= cache.size {
|
|
|
|
popped := cache.list.Front()
|
2018-08-28 21:10:06 -07:00
|
|
|
poppedTxHash := popped.Value.([sha256.Size]byte)
|
|
|
|
delete(cache.map_, poppedTxHash)
|
2018-08-13 03:42:33 -07:00
|
|
|
if popped != nil {
|
|
|
|
cache.list.Remove(popped)
|
|
|
|
}
|
2016-10-07 15:30:17 -07:00
|
|
|
}
|
2018-08-28 21:10:06 -07:00
|
|
|
cache.list.PushBack(txHash)
|
|
|
|
cache.map_[txHash] = cache.list.Back()
|
2016-10-07 15:30:17 -07:00
|
|
|
return true
|
|
|
|
}
|
|
|
|
|
2017-07-13 13:07:04 -04:00
|
|
|
// Remove removes the given tx from the cache.
|
2018-06-19 11:41:16 +04:00
|
|
|
func (cache *mapTxCache) Remove(tx types.Tx) {
|
2016-10-07 15:30:17 -07:00
|
|
|
cache.mtx.Lock()
|
2018-08-28 21:10:06 -07:00
|
|
|
txHash := sha256.Sum256(tx)
|
|
|
|
popped := cache.map_[txHash]
|
|
|
|
delete(cache.map_, txHash)
|
2018-08-13 03:42:33 -07:00
|
|
|
if popped != nil {
|
|
|
|
cache.list.Remove(popped)
|
|
|
|
}
|
|
|
|
|
2016-10-07 15:30:17 -07:00
|
|
|
cache.mtx.Unlock()
|
|
|
|
}
|
2018-06-19 11:41:16 +04:00
|
|
|
|
|
|
|
type nopTxCache struct{}
|
|
|
|
|
|
|
|
var _ txCache = (*nopTxCache)(nil)
|
|
|
|
|
|
|
|
func (nopTxCache) Reset() {}
|
|
|
|
func (nopTxCache) Push(types.Tx) bool { return true }
|
|
|
|
func (nopTxCache) Remove(types.Tx) {}
|