Browse Source

Merge pull request #1824 from tendermint/1762-tm-stops-producing-blocks

Fix emptyPacketMsgSize calc. plus limit max request body size and other mics. changes
pull/1843/head
Ethan Buchman 6 years ago
committed by GitHub
parent
commit
59145eca62
No known key found for this signature in database GPG Key ID: 4AEE18F83AFDEB23
24 changed files with 117 additions and 117 deletions
  1. +7
    -6
      CHANGELOG.md
  2. +1
    -1
      benchmarks/codec_test.go
  3. +3
    -2
      cmd/tendermint/commands/init.go
  4. +19
    -26
      config/config.go
  5. +2
    -7
      config/toml.go
  6. +1
    -1
      consensus/state.go
  7. +1
    -1
      crypto/merkle/doc.go
  8. +4
    -4
      crypto/merkle/simple_tree.go
  9. +1
    -1
      crypto/merkle/simple_tree_test.go
  10. +1
    -1
      docs/running-in-production.md
  11. +1
    -5
      docs/specification/configuration.md
  12. +1
    -1
      evidence/wire.go
  13. +1
    -1
      mempool/reactor.go
  14. +37
    -47
      p2p/conn/connection.go
  15. +2
    -2
      p2p/conn/connection_test.go
  16. +1
    -1
      p2p/switch.go
  17. +1
    -1
      privval/priv_validator.go
  18. +1
    -1
      privval/priv_validator_test.go
  19. +2
    -1
      rpc/client/httpclient.go
  20. +1
    -1
      rpc/core/pipe.go
  21. +7
    -1
      rpc/lib/server/handlers.go
  22. +18
    -2
      rpc/lib/server/http_server.go
  23. +2
    -2
      types/block.go
  24. +2
    -1
      types/params.go

+ 7
- 6
CHANGELOG.md View File

@ -6,7 +6,6 @@
BREAKING CHANGES: BREAKING CHANGES:
- [config] Rename `skip_upnp` to `upnp`, and turn it off by default. - [config] Rename `skip_upnp` to `upnp`, and turn it off by default.
- [config] `MaxPacketMsgPayloadSize` -> `MaxPacketMsgSize`
- [types] Update Amino to v0.10.1 - [types] Update Amino to v0.10.1
* Amino is now fully proto3 compatible for the basic types * Amino is now fully proto3 compatible for the basic types
* JSON-encoded types now use the type name instead of the prefix bytes * JSON-encoded types now use the type name instead of the prefix bytes
@ -25,22 +24,24 @@ FEATURES
- [p2p] Rudimentary IPv6 support - [p2p] Rudimentary IPv6 support
IMPROVEMENT IMPROVEMENT
- [rpc/client] Supports https and wss now.
- [crypto] Make public key size into public constants
- [p2p] Add IPv6 support to peering. - [p2p] Add IPv6 support to peering.
- [rpc/client] Supports https and wss now.
- [crypto] Make public key size into public constants - [crypto] Make public key size into public constants
- [mempool] Log tx hash, not entire tx - [mempool] Log tx hash, not entire tx
- [abci] Merged in github.com/tendermint/abci - [abci] Merged in github.com/tendermint/abci
- [docs] Move from .rst to .md - [docs] Move from .rst to .md
BUG FIXES: BUG FIXES:
- [rpc] Limited number of HTTP/WebSocket connections
- [rpc] Limit maximum number of HTTP/WebSocket connections
(`rpc.max_open_connections`) and gRPC connections (`rpc.max_open_connections`) and gRPC connections
(`rpc.grpc_max_open_connections`). Check out [Running In
Production](https://tendermint.readthedocs.io/en/master/running-in-production.html)
guide if you want to increase them.
(`rpc.grpc_max_open_connections`). Check out "Running In Production" guide if
you want to increase them.
- [rpc] Limit maximum request body size to 1MB (header is limited to 1MB).
- [consensus] Fix a halting bug where `create_empty_blocks=false` - [consensus] Fix a halting bug where `create_empty_blocks=false`
- [p2p] Fix panic in seed mode - [p2p] Fix panic in seed mode
## 0.21.0 ## 0.21.0
*June 21th, 2018* *June 21th, 2018*


+ 1
- 1
benchmarks/codec_test.go View File

@ -6,8 +6,8 @@ import (
"github.com/tendermint/go-amino" "github.com/tendermint/go-amino"
"github.com/tendermint/tendermint/crypto"
proto "github.com/tendermint/tendermint/benchmarks/proto" proto "github.com/tendermint/tendermint/benchmarks/proto"
"github.com/tendermint/tendermint/crypto"
"github.com/tendermint/tendermint/p2p" "github.com/tendermint/tendermint/p2p"
ctypes "github.com/tendermint/tendermint/rpc/core/types" ctypes "github.com/tendermint/tendermint/rpc/core/types"
) )


+ 3
- 2
cmd/tendermint/commands/init.go View File

@ -52,8 +52,9 @@ func initFilesWithConfig(config *cfg.Config) error {
logger.Info("Found genesis file", "path", genFile) logger.Info("Found genesis file", "path", genFile)
} else { } else {
genDoc := types.GenesisDoc{ genDoc := types.GenesisDoc{
ChainID: cmn.Fmt("test-chain-%v", cmn.RandStr(6)),
GenesisTime: time.Now(),
ChainID: cmn.Fmt("test-chain-%v", cmn.RandStr(6)),
GenesisTime: time.Now(),
ConsensusParams: types.DefaultConsensusParams(),
} }
genDoc.Validators = []types.GenesisValidator{{ genDoc.Validators = []types.GenesisValidator{{
PubKey: pv.GetPubKey(), PubKey: pv.GetPubKey(),


+ 19
- 26
config/config.go View File

@ -299,9 +299,8 @@ type P2PConfig struct {
// Time to wait before flushing messages out on the connection, in ms // Time to wait before flushing messages out on the connection, in ms
FlushThrottleTimeout int `mapstructure:"flush_throttle_timeout"` FlushThrottleTimeout int `mapstructure:"flush_throttle_timeout"`
// Maximum size of a message packet, in bytes
// Includes a header, which is ~13 bytes
MaxPacketMsgSize int `mapstructure:"max_packet_msg_size"`
// Maximum size of a message packet payload, in bytes
MaxPacketMsgPayloadSize int `mapstructure:"max_packet_msg_payload_size"`
// Rate at which packets can be sent, in bytes/second // Rate at which packets can be sent, in bytes/second
SendRate int64 `mapstructure:"send_rate"` SendRate int64 `mapstructure:"send_rate"`
@ -340,23 +339,23 @@ type P2PConfig struct {
// DefaultP2PConfig returns a default configuration for the peer-to-peer layer // DefaultP2PConfig returns a default configuration for the peer-to-peer layer
func DefaultP2PConfig() *P2PConfig { func DefaultP2PConfig() *P2PConfig {
return &P2PConfig{ return &P2PConfig{
ListenAddress: "tcp://0.0.0.0:26656",
UPNP: false,
AddrBook: defaultAddrBookPath,
AddrBookStrict: true,
MaxNumPeers: 50,
FlushThrottleTimeout: 100,
MaxPacketMsgSize: 1024, // 1 kB
SendRate: 512000, // 500 kB/s
RecvRate: 512000, // 500 kB/s
PexReactor: true,
SeedMode: false,
AllowDuplicateIP: true, // so non-breaking yet
HandshakeTimeout: 20 * time.Second,
DialTimeout: 3 * time.Second,
TestDialFail: false,
TestFuzz: false,
TestFuzzConfig: DefaultFuzzConnConfig(),
ListenAddress: "tcp://0.0.0.0:26656",
UPNP: false,
AddrBook: defaultAddrBookPath,
AddrBookStrict: true,
MaxNumPeers: 50,
FlushThrottleTimeout: 100,
MaxPacketMsgPayloadSize: 1024, // 1 kB
SendRate: 512000, // 500 kB/s
RecvRate: 512000, // 500 kB/s
PexReactor: true,
SeedMode: false,
AllowDuplicateIP: true, // so non-breaking yet
HandshakeTimeout: 20 * time.Second,
DialTimeout: 3 * time.Second,
TestDialFail: false,
TestFuzz: false,
TestFuzzConfig: DefaultFuzzConnConfig(),
} }
} }
@ -454,10 +453,6 @@ type ConsensusConfig struct {
// Make progress as soon as we have all the precommits (as if TimeoutCommit = 0) // Make progress as soon as we have all the precommits (as if TimeoutCommit = 0)
SkipTimeoutCommit bool `mapstructure:"skip_timeout_commit"` SkipTimeoutCommit bool `mapstructure:"skip_timeout_commit"`
// BlockSize
MaxBlockSizeTxs int `mapstructure:"max_block_size_txs"`
MaxBlockSizeBytes int `mapstructure:"max_block_size_bytes"`
// EmptyBlocks mode and possible interval between empty blocks in seconds // EmptyBlocks mode and possible interval between empty blocks in seconds
CreateEmptyBlocks bool `mapstructure:"create_empty_blocks"` CreateEmptyBlocks bool `mapstructure:"create_empty_blocks"`
CreateEmptyBlocksInterval int `mapstructure:"create_empty_blocks_interval"` CreateEmptyBlocksInterval int `mapstructure:"create_empty_blocks_interval"`
@ -479,8 +474,6 @@ func DefaultConsensusConfig() *ConsensusConfig {
TimeoutPrecommitDelta: 500, TimeoutPrecommitDelta: 500,
TimeoutCommit: 1000, TimeoutCommit: 1000,
SkipTimeoutCommit: false, SkipTimeoutCommit: false,
MaxBlockSizeTxs: 10000,
MaxBlockSizeBytes: 1, // TODO
CreateEmptyBlocks: true, CreateEmptyBlocks: true,
CreateEmptyBlocksInterval: 0, CreateEmptyBlocksInterval: 0,
PeerGossipSleepDuration: 100, PeerGossipSleepDuration: 100,


+ 2
- 7
config/toml.go View File

@ -164,9 +164,8 @@ flush_throttle_timeout = {{ .P2P.FlushThrottleTimeout }}
# Maximum number of peers to connect to # Maximum number of peers to connect to
max_num_peers = {{ .P2P.MaxNumPeers }} max_num_peers = {{ .P2P.MaxNumPeers }}
# Maximum size of a message packet, in bytes
# Includes a header, which is ~13 bytes
max_packet_msg_size = {{ .P2P.MaxPacketMsgSize }}
# Maximum size of a message packet payload, in bytes
max_packet_msg_payload_size = {{ .P2P.MaxPacketMsgPayloadSize }}
# Rate at which packets can be sent, in bytes/second # Rate at which packets can be sent, in bytes/second
send_rate = {{ .P2P.SendRate }} send_rate = {{ .P2P.SendRate }}
@ -217,10 +216,6 @@ timeout_commit = {{ .Consensus.TimeoutCommit }}
# Make progress as soon as we have all the precommits (as if TimeoutCommit = 0) # Make progress as soon as we have all the precommits (as if TimeoutCommit = 0)
skip_timeout_commit = {{ .Consensus.SkipTimeoutCommit }} skip_timeout_commit = {{ .Consensus.SkipTimeoutCommit }}
# BlockSize
max_block_size_txs = {{ .Consensus.MaxBlockSizeTxs }}
max_block_size_bytes = {{ .Consensus.MaxBlockSizeBytes }}
# EmptyBlocks mode and possible interval between empty blocks in seconds # EmptyBlocks mode and possible interval between empty blocks in seconds
create_empty_blocks = {{ .Consensus.CreateEmptyBlocks }} create_empty_blocks = {{ .Consensus.CreateEmptyBlocks }}
create_empty_blocks_interval = {{ .Consensus.CreateEmptyBlocksInterval }} create_empty_blocks_interval = {{ .Consensus.CreateEmptyBlocksInterval }}


+ 1
- 1
consensus/state.go View File

@ -932,7 +932,7 @@ func (cs *ConsensusState) createProposalBlock() (block *types.Block, blockParts
} }
// Mempool validated transactions // Mempool validated transactions
txs := cs.mempool.Reap(cs.config.MaxBlockSizeTxs)
txs := cs.mempool.Reap(cs.state.ConsensusParams.BlockSize.MaxTxs)
block, parts := cs.state.MakeBlock(cs.Height, txs, commit) block, parts := cs.state.MakeBlock(cs.Height, txs, commit)
evidence := cs.evpool.PendingEvidence() evidence := cs.evpool.PendingEvidence()
block.AddEvidence(evidence) block.AddEvidence(evidence)


+ 1
- 1
crypto/merkle/doc.go View File

@ -28,4 +28,4 @@ https://bitcointalk.org/?topic=102395
TODO(ismail): add 2nd pre-image protection or clarify further on how we use this and why this secure. TODO(ismail): add 2nd pre-image protection or clarify further on how we use this and why this secure.
*/ */
package merkle
package merkle

+ 4
- 4
crypto/merkle/simple_tree.go View File

@ -9,12 +9,12 @@ func SimpleHashFromTwoHashes(left, right []byte) []byte {
var hasher = tmhash.New() var hasher = tmhash.New()
err := encodeByteSlice(hasher, left) err := encodeByteSlice(hasher, left)
if err != nil { if err != nil {
panic(err)
}
panic(err)
}
err = encodeByteSlice(hasher, right) err = encodeByteSlice(hasher, right)
if err != nil { if err != nil {
panic(err)
}
panic(err)
}
return hasher.Sum(nil) return hasher.Sum(nil)
} }


+ 1
- 1
crypto/merkle/simple_tree_test.go View File

@ -6,8 +6,8 @@ import (
cmn "github.com/tendermint/tmlibs/common" cmn "github.com/tendermint/tmlibs/common"
. "github.com/tendermint/tmlibs/test" . "github.com/tendermint/tmlibs/test"
"testing"
"github.com/tendermint/tendermint/crypto/tmhash" "github.com/tendermint/tendermint/crypto/tmhash"
"testing"
) )
type testItem []byte type testItem []byte


+ 1
- 1
docs/running-in-production.md View File

@ -21,7 +21,7 @@ to prevent Denial-of-service attacks. You can read more about it
### P2P ### P2P
The core of the Tendermint peer-to-peer system is `MConnection`. Each The core of the Tendermint peer-to-peer system is `MConnection`. Each
connection has `MaxPacketMsgSize`, which is the maximum packet
connection has `MaxPacketMsgPayloadSize`, which is the maximum packet
size and bounded send & receive queues. One can impose restrictions on size and bounded send & receive queues. One can impose restrictions on
send & receive rate per connection (`SendRate`, `RecvRate`). send & receive rate per connection (`SendRate`, `RecvRate`).


+ 1
- 5
docs/specification/configuration.md View File

@ -119,7 +119,7 @@ flush_throttle_timeout = 100
max_num_peers = 50 max_num_peers = 50
# Maximum size of a message packet payload, in bytes # Maximum size of a message packet payload, in bytes
max_msg_packet_payload_size = 1024
max_packet_msg_payload_size = 1024
# Rate at which packets can be sent, in bytes/second # Rate at which packets can be sent, in bytes/second
send_rate = 512000 send_rate = 512000
@ -170,10 +170,6 @@ timeout_commit = 1000
# Make progress as soon as we have all the precommits (as if TimeoutCommit = 0) # Make progress as soon as we have all the precommits (as if TimeoutCommit = 0)
skip_timeout_commit = false skip_timeout_commit = false
# BlockSize
max_block_size_txs = 10000
max_block_size_bytes = 1
# EmptyBlocks mode and possible interval between empty blocks in seconds # EmptyBlocks mode and possible interval between empty blocks in seconds
create_empty_blocks = true create_empty_blocks = true
create_empty_blocks_interval = 0 create_empty_blocks_interval = 0


+ 1
- 1
evidence/wire.go View File

@ -2,8 +2,8 @@ package evidence
import ( import (
"github.com/tendermint/go-amino" "github.com/tendermint/go-amino"
"github.com/tendermint/tendermint/types"
"github.com/tendermint/tendermint/crypto" "github.com/tendermint/tendermint/crypto"
"github.com/tendermint/tendermint/types"
) )
var cdc = amino.NewCodec() var cdc = amino.NewCodec()


+ 1
- 1
mempool/reactor.go View File

@ -5,8 +5,8 @@ import (
"reflect" "reflect"
"time" "time"
abci "github.com/tendermint/tendermint/abci/types"
amino "github.com/tendermint/go-amino" amino "github.com/tendermint/go-amino"
abci "github.com/tendermint/tendermint/abci/types"
"github.com/tendermint/tmlibs/clist" "github.com/tendermint/tmlibs/clist"
"github.com/tendermint/tmlibs/log" "github.com/tendermint/tmlibs/log"


+ 37
- 47
p2p/conn/connection.go View File

@ -18,7 +18,7 @@ import (
) )
const ( const (
defaultMaxPacketMsgSize = 1024
defaultMaxPacketMsgPayloadSize = 1024
numBatchPacketMsgs = 10 numBatchPacketMsgs = 10
minReadBufferSize = 1024 minReadBufferSize = 1024
@ -96,7 +96,7 @@ type MConnection struct {
created time.Time // time of creation created time.Time // time of creation
emptyPacketMsgSize int
_maxPacketMsgSize int
} }
// MConnConfig is a MConnection configuration. // MConnConfig is a MConnection configuration.
@ -105,7 +105,7 @@ type MConnConfig struct {
RecvRate int64 `mapstructure:"recv_rate"` RecvRate int64 `mapstructure:"recv_rate"`
// Maximum payload size // Maximum payload size
MaxPacketMsgSize int `mapstructure:"max_packet_msg_size"`
MaxPacketMsgPayloadSize int `mapstructure:"max_packet_msg_payload_size"`
// Interval to flush writes (throttled) // Interval to flush writes (throttled)
FlushThrottle time.Duration `mapstructure:"flush_throttle"` FlushThrottle time.Duration `mapstructure:"flush_throttle"`
@ -120,12 +120,12 @@ type MConnConfig struct {
// DefaultMConnConfig returns the default config. // DefaultMConnConfig returns the default config.
func DefaultMConnConfig() MConnConfig { func DefaultMConnConfig() MConnConfig {
return MConnConfig{ return MConnConfig{
SendRate: defaultSendRate,
RecvRate: defaultRecvRate,
MaxPacketMsgSize: defaultMaxPacketMsgSize,
FlushThrottle: defaultFlushThrottle,
PingInterval: defaultPingInterval,
PongTimeout: defaultPongTimeout,
SendRate: defaultSendRate,
RecvRate: defaultRecvRate,
MaxPacketMsgPayloadSize: defaultMaxPacketMsgPayloadSize,
FlushThrottle: defaultFlushThrottle,
PingInterval: defaultPingInterval,
PongTimeout: defaultPongTimeout,
} }
} }
@ -146,17 +146,16 @@ func NewMConnectionWithConfig(conn net.Conn, chDescs []*ChannelDescriptor, onRec
} }
mconn := &MConnection{ mconn := &MConnection{
conn: conn,
bufConnReader: bufio.NewReaderSize(conn, minReadBufferSize),
bufConnWriter: bufio.NewWriterSize(conn, minWriteBufferSize),
sendMonitor: flow.New(0, 0),
recvMonitor: flow.New(0, 0),
send: make(chan struct{}, 1),
pong: make(chan struct{}, 1),
onReceive: onReceive,
onError: onError,
config: config,
emptyPacketMsgSize: emptyPacketMsgSize(),
conn: conn,
bufConnReader: bufio.NewReaderSize(conn, minReadBufferSize),
bufConnWriter: bufio.NewWriterSize(conn, minWriteBufferSize),
sendMonitor: flow.New(0, 0),
recvMonitor: flow.New(0, 0),
send: make(chan struct{}, 1),
pong: make(chan struct{}, 1),
onReceive: onReceive,
onError: onError,
config: config,
} }
// Create channels // Create channels
@ -173,6 +172,9 @@ func NewMConnectionWithConfig(conn net.Conn, chDescs []*ChannelDescriptor, onRec
mconn.BaseService = *cmn.NewBaseService(nil, "MConnection", mconn) mconn.BaseService = *cmn.NewBaseService(nil, "MConnection", mconn)
// maxPacketMsgSize() is a bit heavy, so call just once
mconn._maxPacketMsgSize = mconn.maxPacketMsgSize()
return mconn return mconn
} }
@ -397,7 +399,7 @@ func (c *MConnection) sendSomePacketMsgs() bool {
// Block until .sendMonitor says we can write. // Block until .sendMonitor says we can write.
// Once we're ready we send more than we asked for, // Once we're ready we send more than we asked for,
// but amortized it should even out. // but amortized it should even out.
c.sendMonitor.Limit(c.config.MaxPacketMsgSize, atomic.LoadInt64(&c.config.SendRate), true)
c.sendMonitor.Limit(c._maxPacketMsgSize, atomic.LoadInt64(&c.config.SendRate), true)
// Now send some PacketMsgs. // Now send some PacketMsgs.
for i := 0; i < numBatchPacketMsgs; i++ { for i := 0; i < numBatchPacketMsgs; i++ {
@ -455,7 +457,7 @@ func (c *MConnection) recvRoutine() {
FOR_LOOP: FOR_LOOP:
for { for {
// Block until .recvMonitor says we can read. // Block until .recvMonitor says we can read.
c.recvMonitor.Limit(c.config.MaxPacketMsgSize, atomic.LoadInt64(&c.config.RecvRate), true)
c.recvMonitor.Limit(c._maxPacketMsgSize, atomic.LoadInt64(&c.config.RecvRate), true)
// Peek into bufConnReader for debugging // Peek into bufConnReader for debugging
/* /*
@ -475,7 +477,7 @@ FOR_LOOP:
var packet Packet var packet Packet
var _n int64 var _n int64
var err error var err error
_n, err = cdc.UnmarshalBinaryReader(c.bufConnReader, &packet, int64(c.config.MaxPacketMsgSize))
_n, err = cdc.UnmarshalBinaryReader(c.bufConnReader, &packet, int64(c._maxPacketMsgSize))
c.recvMonitor.Update(int(_n)) c.recvMonitor.Update(int(_n))
if err != nil { if err != nil {
if c.IsRunning() { if c.IsRunning() {
@ -548,6 +550,16 @@ func (c *MConnection) stopPongTimer() {
} }
} }
// maxPacketMsgSize returns a maximum size of PacketMsg, including the overhead
// of amino encoding.
func (c *MConnection) maxPacketMsgSize() int {
return len(cdc.MustMarshalBinary(PacketMsg{
ChannelID: 0x01,
EOF: 1,
Bytes: make([]byte, c.config.MaxPacketMsgPayloadSize),
})) + 10 // leave room for changes in amino
}
type ConnectionStatus struct { type ConnectionStatus struct {
Duration time.Duration Duration time.Duration
SendMonitor flow.Status SendMonitor flow.Status
@ -631,7 +643,7 @@ func newChannel(conn *MConnection, desc ChannelDescriptor) *Channel {
desc: desc, desc: desc,
sendQueue: make(chan []byte, desc.SendQueueCapacity), sendQueue: make(chan []byte, desc.SendQueueCapacity),
recving: make([]byte, 0, desc.RecvBufferCapacity), recving: make([]byte, 0, desc.RecvBufferCapacity),
maxPacketMsgPayloadSize: conn.config.MaxPacketMsgSize,
maxPacketMsgPayloadSize: conn.config.MaxPacketMsgPayloadSize,
} }
} }
@ -694,7 +706,7 @@ func (ch *Channel) isSendPending() bool {
func (ch *Channel) nextPacketMsg() PacketMsg { func (ch *Channel) nextPacketMsg() PacketMsg {
packet := PacketMsg{} packet := PacketMsg{}
packet.ChannelID = byte(ch.desc.ID) packet.ChannelID = byte(ch.desc.ID)
maxSize := ch.maxPacketMsgPayloadSize - ch.conn.emptyPacketMsgSize
maxSize := ch.maxPacketMsgPayloadSize
packet.Bytes = ch.sending[:cmn.MinInt(maxSize, len(ch.sending))] packet.Bytes = ch.sending[:cmn.MinInt(maxSize, len(ch.sending))]
if len(ch.sending) <= maxSize { if len(ch.sending) <= maxSize {
packet.EOF = byte(0x01) packet.EOF = byte(0x01)
@ -780,25 +792,3 @@ type PacketMsg struct {
func (mp PacketMsg) String() string { func (mp PacketMsg) String() string {
return fmt.Sprintf("PacketMsg{%X:%X T:%X}", mp.ChannelID, mp.Bytes, mp.EOF) return fmt.Sprintf("PacketMsg{%X:%X T:%X}", mp.ChannelID, mp.Bytes, mp.EOF)
} }
// - Uvarint length of MustMarshalBinary(packet) = 1 or 2 bytes
// (as long as it's less than 16,384 bytes)
// - Prefix bytes = 4 bytes
// - ChannelID field key + byte = 2 bytes
// - EOF field key + byte = 2 bytes
// - Bytes field key = 1 bytes
// - Uvarint length of MustMarshalBinary(bytes) = 1 or 2 bytes
// - Struct terminator = 1 byte
// = up to 14 bytes overhead for the packet.
func emptyPacketMsgSize() int {
emptyPacketMsgSize := len(cdc.MustMarshalBinary(PacketMsg{
ChannelID: 0x01,
EOF: 1,
Bytes: make([]byte, 1),
}))
// -1 byte of data
// +1 byte because uvarint length of MustMarshalBinary(bytes) will be 2 bytes for big packets
// +1 byte because uvarint length of MustMarshalBinary(packet) will be 2 bytes for big packets
return emptyPacketMsgSize - 1 + 1 + 1
}

+ 2
- 2
p2p/conn/connection_test.go View File

@ -426,7 +426,7 @@ func TestMConnectionReadErrorLongMessage(t *testing.T) {
var packet = PacketMsg{ var packet = PacketMsg{
ChannelID: 0x01, ChannelID: 0x01,
EOF: 1, EOF: 1,
Bytes: make([]byte, mconnClient.config.MaxPacketMsgSize-emptyPacketMsgSize()),
Bytes: make([]byte, mconnClient.config.MaxPacketMsgPayloadSize),
} }
_, err = cdc.MarshalBinaryWriter(buf, packet) _, err = cdc.MarshalBinaryWriter(buf, packet)
assert.Nil(t, err) assert.Nil(t, err)
@ -440,7 +440,7 @@ func TestMConnectionReadErrorLongMessage(t *testing.T) {
packet = PacketMsg{ packet = PacketMsg{
ChannelID: 0x01, ChannelID: 0x01,
EOF: 1, EOF: 1,
Bytes: make([]byte, mconnClient.config.MaxPacketMsgSize+1),
Bytes: make([]byte, mconnClient.config.MaxPacketMsgPayloadSize+100),
} }
_, err = cdc.MarshalBinaryWriter(buf, packet) _, err = cdc.MarshalBinaryWriter(buf, packet)
assert.Nil(t, err) assert.Nil(t, err)


+ 1
- 1
p2p/switch.go View File

@ -100,7 +100,7 @@ func NewSwitch(cfg *config.P2PConfig, options ...SwitchOption) *Switch {
mConfig.FlushThrottle = time.Duration(cfg.FlushThrottleTimeout) * time.Millisecond mConfig.FlushThrottle = time.Duration(cfg.FlushThrottleTimeout) * time.Millisecond
mConfig.SendRate = cfg.SendRate mConfig.SendRate = cfg.SendRate
mConfig.RecvRate = cfg.RecvRate mConfig.RecvRate = cfg.RecvRate
mConfig.MaxPacketMsgSize = cfg.MaxPacketMsgSize
mConfig.MaxPacketMsgPayloadSize = cfg.MaxPacketMsgPayloadSize
sw.mConfig = mConfig sw.mConfig = mConfig


+ 1
- 1
privval/priv_validator.go View File

@ -287,7 +287,7 @@ func (pv *FilePV) saveSigned(height int64, round int, step int8,
func (pv *FilePV) SignHeartbeat(chainID string, heartbeat *types.Heartbeat) error { func (pv *FilePV) SignHeartbeat(chainID string, heartbeat *types.Heartbeat) error {
pv.mtx.Lock() pv.mtx.Lock()
defer pv.mtx.Unlock() defer pv.mtx.Unlock()
sig, err:= pv.PrivKey.Sign(heartbeat.SignBytes(chainID))
sig, err := pv.PrivKey.Sign(heartbeat.SignBytes(chainID))
if err != nil { if err != nil {
return err return err
} }


+ 1
- 1
privval/priv_validator_test.go View File

@ -9,8 +9,8 @@ import (
"github.com/stretchr/testify/assert" "github.com/stretchr/testify/assert"
"github.com/stretchr/testify/require" "github.com/stretchr/testify/require"
"github.com/tendermint/tendermint/types"
"github.com/tendermint/tendermint/crypto" "github.com/tendermint/tendermint/crypto"
"github.com/tendermint/tendermint/types"
cmn "github.com/tendermint/tmlibs/common" cmn "github.com/tendermint/tmlibs/common"
) )


+ 2
- 1
rpc/client/httpclient.go View File

@ -29,12 +29,13 @@ type HTTP struct {
*WSEvents *WSEvents
} }
// New takes a remote endpoint in the form tcp://<host>:<port>
// NewHTTP takes a remote endpoint in the form tcp://<host>:<port>
// and the websocket path (which always seems to be "/websocket") // and the websocket path (which always seems to be "/websocket")
func NewHTTP(remote, wsEndpoint string) *HTTP { func NewHTTP(remote, wsEndpoint string) *HTTP {
rc := rpcclient.NewJSONRPCClient(remote) rc := rpcclient.NewJSONRPCClient(remote)
cdc := rc.Codec() cdc := rc.Codec()
ctypes.RegisterAmino(cdc) ctypes.RegisterAmino(cdc)
rc.SetCodec(cdc)
return &HTTP{ return &HTTP{
rpc: rc, rpc: rc,


+ 1
- 1
rpc/core/pipe.go View File

@ -3,8 +3,8 @@ package core
import ( import (
"time" "time"
crypto "github.com/tendermint/tendermint/crypto"
"github.com/tendermint/tendermint/consensus" "github.com/tendermint/tendermint/consensus"
crypto "github.com/tendermint/tendermint/crypto"
"github.com/tendermint/tendermint/p2p" "github.com/tendermint/tendermint/p2p"
"github.com/tendermint/tendermint/proxy" "github.com/tendermint/tendermint/proxy"
sm "github.com/tendermint/tendermint/state" sm "github.com/tendermint/tendermint/state"


+ 7
- 1
rpc/lib/server/handlers.go View File

@ -406,7 +406,13 @@ type wsConnection struct {
// description of how to configure ping period and pong wait time. NOTE: if the // description of how to configure ping period and pong wait time. NOTE: if the
// write buffer is full, pongs may be dropped, which may cause clients to // write buffer is full, pongs may be dropped, which may cause clients to
// disconnect. see https://github.com/gorilla/websocket/issues/97 // disconnect. see https://github.com/gorilla/websocket/issues/97
func NewWSConnection(baseConn *websocket.Conn, funcMap map[string]*RPCFunc, cdc *amino.Codec, options ...func(*wsConnection)) *wsConnection {
func NewWSConnection(
baseConn *websocket.Conn,
funcMap map[string]*RPCFunc,
cdc *amino.Codec,
options ...func(*wsConnection),
) *wsConnection {
baseConn.SetReadLimit(maxBodyBytes)
wsc := &wsConnection{ wsc := &wsConnection{
remoteAddr: baseConn.RemoteAddr().String(), remoteAddr: baseConn.RemoteAddr().String(),
baseConn: baseConn, baseConn: baseConn,


+ 18
- 2
rpc/lib/server/http_server.go View File

@ -23,6 +23,12 @@ type Config struct {
MaxOpenConnections int MaxOpenConnections int
} }
const (
// maxBodyBytes controls the maximum number of bytes the
// server will read parsing the request body.
maxBodyBytes = int64(1000000) // 1MB
)
// StartHTTPServer starts an HTTP server on listenAddr with the given handler. // StartHTTPServer starts an HTTP server on listenAddr with the given handler.
// It wraps handler with RecoverAndLogHandler. // It wraps handler with RecoverAndLogHandler.
func StartHTTPServer( func StartHTTPServer(
@ -53,7 +59,7 @@ func StartHTTPServer(
go func() { go func() {
err := http.Serve( err := http.Serve(
listener, listener,
RecoverAndLogHandler(handler, logger),
RecoverAndLogHandler(maxBytesHandler{h: handler, n: maxBodyBytes}, logger),
) )
logger.Error("RPC HTTP server stopped", "err", err) logger.Error("RPC HTTP server stopped", "err", err)
}() }()
@ -99,7 +105,7 @@ func StartHTTPAndTLSServer(
go func() { go func() {
err := http.ServeTLS( err := http.ServeTLS(
listener, listener,
RecoverAndLogHandler(handler, logger),
RecoverAndLogHandler(maxBytesHandler{h: handler, n: maxBodyBytes}, logger),
certFile, certFile,
keyFile, keyFile,
) )
@ -202,3 +208,13 @@ func (w *ResponseWriterWrapper) WriteHeader(status int) {
func (w *ResponseWriterWrapper) Hijack() (net.Conn, *bufio.ReadWriter, error) { func (w *ResponseWriterWrapper) Hijack() (net.Conn, *bufio.ReadWriter, error) {
return w.ResponseWriter.(http.Hijacker).Hijack() return w.ResponseWriter.(http.Hijacker).Hijack()
} }
type maxBytesHandler struct {
h http.Handler
n int64
}
func (h maxBytesHandler) ServeHTTP(w http.ResponseWriter, r *http.Request) {
r.Body = http.MaxBytesReader(w, r.Body, h.n)
h.h.ServeHTTP(w, r)
}

+ 2
- 2
types/block.go View File

@ -464,7 +464,7 @@ func (data *Data) StringIndented(indent string) string {
txStrings[i] = fmt.Sprintf("... (%v total)", len(data.Txs)) txStrings[i] = fmt.Sprintf("... (%v total)", len(data.Txs))
break break
} }
txStrings[i] = fmt.Sprintf("Tx:%v", tx)
txStrings[i] = fmt.Sprintf("%X (%d bytes)", tx.Hash(), len(tx))
} }
return fmt.Sprintf(`Data{ return fmt.Sprintf(`Data{
%s %v %s %v
@ -504,7 +504,7 @@ func (data *EvidenceData) StringIndented(indent string) string {
} }
evStrings[i] = fmt.Sprintf("Evidence:%v", ev) evStrings[i] = fmt.Sprintf("Evidence:%v", ev)
} }
return fmt.Sprintf(`Data{
return fmt.Sprintf(`EvidenceData{
%s %v %s %v
%s}#%v`, %s}#%v`,
indent, strings.Join(evStrings, "\n"+indent+" "), indent, strings.Join(evStrings, "\n"+indent+" "),


+ 2
- 1
types/params.go View File

@ -7,6 +7,7 @@ import (
) )
const ( const (
// MaxBlockSizeBytes is the maximum permitted size of the blocks.
MaxBlockSizeBytes = 104857600 // 100MB MaxBlockSizeBytes = 104857600 // 100MB
) )
@ -56,7 +57,7 @@ func DefaultConsensusParams() *ConsensusParams {
func DefaultBlockSize() BlockSize { func DefaultBlockSize() BlockSize {
return BlockSize{ return BlockSize{
MaxBytes: 22020096, // 21MB MaxBytes: 22020096, // 21MB
MaxTxs: 100000,
MaxTxs: 10000,
MaxGas: -1, MaxGas: -1,
} }
} }


Loading…
Cancel
Save