You can not select more than 25 topics Topics must start with a letter or number, can include dashes ('-') and can be up to 35 characters long.

790 lines
21 KiB

p2p: implement new Transport interface (#5791) This implements a new `Transport` interface and related types for the P2P refactor in #5670. Previously, `conn.MConnection` was very tightly coupled to the `Peer` implementation -- in order to allow alternative non-multiplexed transports (e.g. QUIC), MConnection has now been moved below the `Transport` interface, as `MConnTransport`, and decoupled from the peer. Since the `p2p` package is not covered by our Go API stability, this is not considered a breaking change, and not listed in the changelog. The initial approach was to implement the new interface in its final form (which also involved possible protocol changes, see https://github.com/tendermint/spec/pull/227). However, it turned out that this would require a large amount of changes to existing P2P code because of the previous tight coupling between `Peer` and `MConnection` and the reliance on subtleties in the MConnection behavior. Instead, I have broadened the `Transport` interface to expose much of the existing MConnection interface, preserved much of the existing MConnection logic and behavior in the transport implementation, and tried to make as few changes to the rest of the P2P stack as possible. We will instead reduce this interface gradually as we refactor other parts of the P2P stack. The low-level transport code and protocol (e.g. MConnection, SecretConnection and so on) has not been significantly changed, and refactoring this is not a priority until we come up with a plan for QUIC adoption, as we may end up discarding the MConnection code entirely. There are no tests of the new `MConnTransport`, as this code is likely to evolve as we proceed with the P2P refactor, but tests should be added before a final release. The E2E tests are sufficient for basic validation in the meanwhile.
4 years ago
p2p: implement new Transport interface (#5791) This implements a new `Transport` interface and related types for the P2P refactor in #5670. Previously, `conn.MConnection` was very tightly coupled to the `Peer` implementation -- in order to allow alternative non-multiplexed transports (e.g. QUIC), MConnection has now been moved below the `Transport` interface, as `MConnTransport`, and decoupled from the peer. Since the `p2p` package is not covered by our Go API stability, this is not considered a breaking change, and not listed in the changelog. The initial approach was to implement the new interface in its final form (which also involved possible protocol changes, see https://github.com/tendermint/spec/pull/227). However, it turned out that this would require a large amount of changes to existing P2P code because of the previous tight coupling between `Peer` and `MConnection` and the reliance on subtleties in the MConnection behavior. Instead, I have broadened the `Transport` interface to expose much of the existing MConnection interface, preserved much of the existing MConnection logic and behavior in the transport implementation, and tried to make as few changes to the rest of the P2P stack as possible. We will instead reduce this interface gradually as we refactor other parts of the P2P stack. The low-level transport code and protocol (e.g. MConnection, SecretConnection and so on) has not been significantly changed, and refactoring this is not a priority until we come up with a plan for QUIC adoption, as we may end up discarding the MConnection code entirely. There are no tests of the new `MConnTransport`, as this code is likely to evolve as we proceed with the P2P refactor, but tests should be added before a final release. The E2E tests are sufficient for basic validation in the meanwhile.
4 years ago
  1. package conn
  2. import (
  3. "bufio"
  4. "context"
  5. "errors"
  6. "fmt"
  7. "io"
  8. "math"
  9. "net"
  10. "reflect"
  11. "runtime/debug"
  12. "sync"
  13. "sync/atomic"
  14. "time"
  15. "github.com/gogo/protobuf/proto"
  16. "github.com/tendermint/tendermint/internal/libs/flowrate"
  17. "github.com/tendermint/tendermint/internal/libs/protoio"
  18. "github.com/tendermint/tendermint/internal/libs/timer"
  19. "github.com/tendermint/tendermint/libs/log"
  20. tmmath "github.com/tendermint/tendermint/libs/math"
  21. "github.com/tendermint/tendermint/libs/service"
  22. tmp2p "github.com/tendermint/tendermint/proto/tendermint/p2p"
  23. )
  24. const (
  25. // mirrors MaxPacketMsgPayloadSize from config/config.go
  26. defaultMaxPacketMsgPayloadSize = 1400
  27. numBatchPacketMsgs = 10
  28. minReadBufferSize = 1024
  29. minWriteBufferSize = 65536
  30. updateStats = 2 * time.Second
  31. // some of these defaults are written in the user config
  32. // flushThrottle, sendRate, recvRate
  33. // TODO: remove values present in config
  34. defaultFlushThrottle = 100 * time.Millisecond
  35. defaultSendQueueCapacity = 1
  36. defaultRecvBufferCapacity = 4096
  37. defaultRecvMessageCapacity = 22020096 // 21MB
  38. defaultSendRate = int64(512000) // 500KB/s
  39. defaultRecvRate = int64(512000) // 500KB/s
  40. defaultSendTimeout = 10 * time.Second
  41. defaultPingInterval = 60 * time.Second
  42. defaultPongTimeout = 45 * time.Second
  43. )
  44. type receiveCbFunc func(ctx context.Context, chID ChannelID, msgBytes []byte)
  45. type errorCbFunc func(context.Context, interface{})
  46. /*
  47. Each peer has one `MConnection` (multiplex connection) instance.
  48. __multiplex__ *noun* a system or signal involving simultaneous transmission of
  49. several messages along a single channel of communication.
  50. Each `MConnection` handles message transmission on multiple abstract communication
  51. `Channel`s. Each channel has a globally unique byte id.
  52. The byte id and the relative priorities of each `Channel` are configured upon
  53. initialization of the connection.
  54. There are two methods for sending messages:
  55. func (m MConnection) Send(chID byte, msgBytes []byte) bool {}
  56. `Send(chID, msgBytes)` is a blocking call that waits until `msg` is
  57. successfully queued for the channel with the given id byte `chID`, or until the
  58. request times out. The message `msg` is serialized using Protobuf.
  59. Inbound message bytes are handled with an onReceive callback function.
  60. */
  61. type MConnection struct {
  62. service.BaseService
  63. logger log.Logger
  64. conn net.Conn
  65. bufConnReader *bufio.Reader
  66. bufConnWriter *bufio.Writer
  67. sendMonitor *flowrate.Monitor
  68. recvMonitor *flowrate.Monitor
  69. send chan struct{}
  70. pong chan struct{}
  71. channels []*channel
  72. channelsIdx map[ChannelID]*channel
  73. onReceive receiveCbFunc
  74. onError errorCbFunc
  75. errored uint32
  76. config MConnConfig
  77. // Closing quitSendRoutine will cause the sendRoutine to eventually quit.
  78. // doneSendRoutine is closed when the sendRoutine actually quits.
  79. quitSendRoutine chan struct{}
  80. doneSendRoutine chan struct{}
  81. // Closing quitRecvRouting will cause the recvRouting to eventually quit.
  82. quitRecvRoutine chan struct{}
  83. // used to ensure FlushStop and OnStop
  84. // are safe to call concurrently.
  85. stopMtx sync.Mutex
  86. cancel context.CancelFunc
  87. flushTimer *timer.ThrottleTimer // flush writes as necessary but throttled.
  88. pingTimer *time.Ticker // send pings periodically
  89. // close conn if pong is not received in pongTimeout
  90. pongTimer *time.Timer
  91. pongTimeoutCh chan bool // true - timeout, false - peer sent pong
  92. chStatsTimer *time.Ticker // update channel stats periodically
  93. created time.Time // time of creation
  94. _maxPacketMsgSize int
  95. }
  96. // MConnConfig is a MConnection configuration.
  97. type MConnConfig struct {
  98. SendRate int64 `mapstructure:"send_rate"`
  99. RecvRate int64 `mapstructure:"recv_rate"`
  100. // Maximum payload size
  101. MaxPacketMsgPayloadSize int `mapstructure:"max_packet_msg_payload_size"`
  102. // Interval to flush writes (throttled)
  103. FlushThrottle time.Duration `mapstructure:"flush_throttle"`
  104. // Interval to send pings
  105. PingInterval time.Duration `mapstructure:"ping_interval"`
  106. // Maximum wait time for pongs
  107. PongTimeout time.Duration `mapstructure:"pong_timeout"`
  108. }
  109. // DefaultMConnConfig returns the default config.
  110. func DefaultMConnConfig() MConnConfig {
  111. return MConnConfig{
  112. SendRate: defaultSendRate,
  113. RecvRate: defaultRecvRate,
  114. MaxPacketMsgPayloadSize: defaultMaxPacketMsgPayloadSize,
  115. FlushThrottle: defaultFlushThrottle,
  116. PingInterval: defaultPingInterval,
  117. PongTimeout: defaultPongTimeout,
  118. }
  119. }
  120. // NewMConnection wraps net.Conn and creates multiplex connection
  121. func NewMConnection(
  122. logger log.Logger,
  123. conn net.Conn,
  124. chDescs []*ChannelDescriptor,
  125. onReceive receiveCbFunc,
  126. onError errorCbFunc,
  127. ) *MConnection {
  128. return NewMConnectionWithConfig(
  129. logger,
  130. conn,
  131. chDescs,
  132. onReceive,
  133. onError,
  134. DefaultMConnConfig())
  135. }
  136. // NewMConnectionWithConfig wraps net.Conn and creates multiplex connection with a config
  137. func NewMConnectionWithConfig(
  138. logger log.Logger,
  139. conn net.Conn,
  140. chDescs []*ChannelDescriptor,
  141. onReceive receiveCbFunc,
  142. onError errorCbFunc,
  143. config MConnConfig,
  144. ) *MConnection {
  145. if config.PongTimeout >= config.PingInterval {
  146. panic("pongTimeout must be less than pingInterval (otherwise, next ping will reset pong timer)")
  147. }
  148. mconn := &MConnection{
  149. logger: logger,
  150. conn: conn,
  151. bufConnReader: bufio.NewReaderSize(conn, minReadBufferSize),
  152. bufConnWriter: bufio.NewWriterSize(conn, minWriteBufferSize),
  153. sendMonitor: flowrate.New(0, 0),
  154. recvMonitor: flowrate.New(0, 0),
  155. send: make(chan struct{}, 1),
  156. pong: make(chan struct{}, 1),
  157. onReceive: onReceive,
  158. onError: onError,
  159. config: config,
  160. created: time.Now(),
  161. cancel: func() {},
  162. }
  163. mconn.BaseService = *service.NewBaseService(logger, "MConnection", mconn)
  164. // Create channels
  165. var channelsIdx = map[ChannelID]*channel{}
  166. var channels = []*channel{}
  167. for _, desc := range chDescs {
  168. channel := newChannel(mconn, *desc)
  169. channelsIdx[channel.desc.ID] = channel
  170. channels = append(channels, channel)
  171. }
  172. mconn.channels = channels
  173. mconn.channelsIdx = channelsIdx
  174. // maxPacketMsgSize() is a bit heavy, so call just once
  175. mconn._maxPacketMsgSize = mconn.maxPacketMsgSize()
  176. return mconn
  177. }
  178. // OnStart implements BaseService
  179. func (c *MConnection) OnStart(ctx context.Context) error {
  180. c.flushTimer = timer.NewThrottleTimer("flush", c.config.FlushThrottle)
  181. c.pingTimer = time.NewTicker(c.config.PingInterval)
  182. c.pongTimeoutCh = make(chan bool, 1)
  183. c.chStatsTimer = time.NewTicker(updateStats)
  184. c.quitSendRoutine = make(chan struct{})
  185. c.doneSendRoutine = make(chan struct{})
  186. c.quitRecvRoutine = make(chan struct{})
  187. go c.sendRoutine(ctx)
  188. go c.recvRoutine(ctx)
  189. return nil
  190. }
  191. // stopServices stops the BaseService and timers and closes the quitSendRoutine.
  192. // if the quitSendRoutine was already closed, it returns true, otherwise it returns false.
  193. // It uses the stopMtx to ensure only one of FlushStop and OnStop can do this at a time.
  194. func (c *MConnection) stopServices() (alreadyStopped bool) {
  195. c.stopMtx.Lock()
  196. defer c.stopMtx.Unlock()
  197. select {
  198. case <-c.quitSendRoutine:
  199. // already quit
  200. return true
  201. default:
  202. }
  203. select {
  204. case <-c.quitRecvRoutine:
  205. // already quit
  206. return true
  207. default:
  208. }
  209. c.flushTimer.Stop()
  210. c.pingTimer.Stop()
  211. c.chStatsTimer.Stop()
  212. // inform the recvRouting that we are shutting down
  213. close(c.quitRecvRoutine)
  214. close(c.quitSendRoutine)
  215. return false
  216. }
  217. // OnStop implements BaseService
  218. func (c *MConnection) OnStop() {
  219. if c.stopServices() {
  220. return
  221. }
  222. c.conn.Close()
  223. // We can't close pong safely here because
  224. // recvRoutine may write to it after we've stopped.
  225. // Though it doesn't need to get closed at all,
  226. // we close it @ recvRoutine.
  227. }
  228. func (c *MConnection) String() string {
  229. return fmt.Sprintf("MConn{%v}", c.conn.RemoteAddr())
  230. }
  231. func (c *MConnection) flush() {
  232. c.logger.Debug("Flush", "conn", c)
  233. err := c.bufConnWriter.Flush()
  234. if err != nil {
  235. c.logger.Debug("MConnection flush failed", "err", err)
  236. }
  237. }
  238. // Catch panics, usually caused by remote disconnects.
  239. func (c *MConnection) _recover(ctx context.Context) {
  240. if r := recover(); r != nil {
  241. c.logger.Error("MConnection panicked", "err", r, "stack", string(debug.Stack()))
  242. c.stopForError(ctx, fmt.Errorf("recovered from panic: %v", r))
  243. }
  244. }
  245. func (c *MConnection) stopForError(ctx context.Context, r interface{}) {
  246. if err := c.Stop(); err != nil {
  247. c.logger.Error("Error stopping connection", "err", err)
  248. }
  249. if atomic.CompareAndSwapUint32(&c.errored, 0, 1) {
  250. if c.onError != nil {
  251. c.onError(ctx, r)
  252. }
  253. }
  254. }
  255. // Queues a message to be sent to channel.
  256. func (c *MConnection) Send(chID ChannelID, msgBytes []byte) bool {
  257. if !c.IsRunning() {
  258. return false
  259. }
  260. c.logger.Debug("Send", "channel", chID, "conn", c, "msgBytes", msgBytes)
  261. // Send message to channel.
  262. channel, ok := c.channelsIdx[chID]
  263. if !ok {
  264. c.logger.Error(fmt.Sprintf("Cannot send bytes, unknown channel %X", chID))
  265. return false
  266. }
  267. success := channel.sendBytes(msgBytes)
  268. if success {
  269. // Wake up sendRoutine if necessary
  270. select {
  271. case c.send <- struct{}{}:
  272. default:
  273. }
  274. } else {
  275. c.logger.Debug("Send failed", "channel", chID, "conn", c, "msgBytes", msgBytes)
  276. }
  277. return success
  278. }
  279. // sendRoutine polls for packets to send from channels.
  280. func (c *MConnection) sendRoutine(ctx context.Context) {
  281. defer c._recover(ctx)
  282. protoWriter := protoio.NewDelimitedWriter(c.bufConnWriter)
  283. FOR_LOOP:
  284. for {
  285. var _n int
  286. var err error
  287. SELECTION:
  288. select {
  289. case <-c.flushTimer.Ch:
  290. // NOTE: flushTimer.Set() must be called every time
  291. // something is written to .bufConnWriter.
  292. c.flush()
  293. case <-c.chStatsTimer.C:
  294. for _, channel := range c.channels {
  295. channel.updateStats()
  296. }
  297. case <-c.pingTimer.C:
  298. c.logger.Debug("Send Ping")
  299. _n, err = protoWriter.WriteMsg(mustWrapPacket(&tmp2p.PacketPing{}))
  300. if err != nil {
  301. c.logger.Error("Failed to send PacketPing", "err", err)
  302. break SELECTION
  303. }
  304. c.sendMonitor.Update(_n)
  305. c.logger.Debug("Starting pong timer", "dur", c.config.PongTimeout)
  306. c.pongTimer = time.AfterFunc(c.config.PongTimeout, func() {
  307. select {
  308. case c.pongTimeoutCh <- true:
  309. default:
  310. }
  311. })
  312. c.flush()
  313. case timeout := <-c.pongTimeoutCh:
  314. if timeout {
  315. c.logger.Debug("Pong timeout")
  316. err = errors.New("pong timeout")
  317. } else {
  318. c.stopPongTimer()
  319. }
  320. case <-c.pong:
  321. c.logger.Debug("Send Pong")
  322. _n, err = protoWriter.WriteMsg(mustWrapPacket(&tmp2p.PacketPong{}))
  323. if err != nil {
  324. c.logger.Error("Failed to send PacketPong", "err", err)
  325. break SELECTION
  326. }
  327. c.sendMonitor.Update(_n)
  328. c.flush()
  329. case <-ctx.Done():
  330. break FOR_LOOP
  331. case <-c.quitSendRoutine:
  332. break FOR_LOOP
  333. case <-c.send:
  334. // Send some PacketMsgs
  335. eof := c.sendSomePacketMsgs(ctx)
  336. if !eof {
  337. // Keep sendRoutine awake.
  338. select {
  339. case c.send <- struct{}{}:
  340. default:
  341. }
  342. }
  343. }
  344. if !c.IsRunning() {
  345. break FOR_LOOP
  346. }
  347. if err != nil {
  348. c.logger.Error("Connection failed @ sendRoutine", "conn", c, "err", err)
  349. c.stopForError(ctx, err)
  350. break FOR_LOOP
  351. }
  352. }
  353. // Cleanup
  354. c.stopPongTimer()
  355. close(c.doneSendRoutine)
  356. }
  357. // Returns true if messages from channels were exhausted.
  358. // Blocks in accordance to .sendMonitor throttling.
  359. func (c *MConnection) sendSomePacketMsgs(ctx context.Context) bool {
  360. // Block until .sendMonitor says we can write.
  361. // Once we're ready we send more than we asked for,
  362. // but amortized it should even out.
  363. c.sendMonitor.Limit(c._maxPacketMsgSize, atomic.LoadInt64(&c.config.SendRate), true)
  364. // Now send some PacketMsgs.
  365. for i := 0; i < numBatchPacketMsgs; i++ {
  366. if c.sendPacketMsg(ctx) {
  367. return true
  368. }
  369. }
  370. return false
  371. }
  372. // Returns true if messages from channels were exhausted.
  373. func (c *MConnection) sendPacketMsg(ctx context.Context) bool {
  374. // Choose a channel to create a PacketMsg from.
  375. // The chosen channel will be the one whose recentlySent/priority is the least.
  376. var leastRatio float32 = math.MaxFloat32
  377. var leastChannel *channel
  378. for _, channel := range c.channels {
  379. // If nothing to send, skip this channel
  380. if !channel.isSendPending() {
  381. continue
  382. }
  383. // Get ratio, and keep track of lowest ratio.
  384. ratio := float32(channel.recentlySent) / float32(channel.desc.Priority)
  385. if ratio < leastRatio {
  386. leastRatio = ratio
  387. leastChannel = channel
  388. }
  389. }
  390. // Nothing to send?
  391. if leastChannel == nil {
  392. return true
  393. }
  394. // c.logger.Info("Found a msgPacket to send")
  395. // Make & send a PacketMsg from this channel
  396. _n, err := leastChannel.writePacketMsgTo(c.bufConnWriter)
  397. if err != nil {
  398. c.logger.Error("Failed to write PacketMsg", "err", err)
  399. c.stopForError(ctx, err)
  400. return true
  401. }
  402. c.sendMonitor.Update(_n)
  403. c.flushTimer.Set()
  404. return false
  405. }
  406. // recvRoutine reads PacketMsgs and reconstructs the message using the channels' "recving" buffer.
  407. // After a whole message has been assembled, it's pushed to onReceive().
  408. // Blocks depending on how the connection is throttled.
  409. // Otherwise, it never blocks.
  410. func (c *MConnection) recvRoutine(ctx context.Context) {
  411. defer c._recover(ctx)
  412. protoReader := protoio.NewDelimitedReader(c.bufConnReader, c._maxPacketMsgSize)
  413. FOR_LOOP:
  414. for {
  415. // Block until .recvMonitor says we can read.
  416. c.recvMonitor.Limit(c._maxPacketMsgSize, atomic.LoadInt64(&c.config.RecvRate), true)
  417. // Peek into bufConnReader for debugging
  418. /*
  419. if numBytes := c.bufConnReader.Buffered(); numBytes > 0 {
  420. bz, err := c.bufConnReader.Peek(tmmath.MinInt(numBytes, 100))
  421. if err == nil {
  422. // return
  423. } else {
  424. c.logger.Debug("Error peeking connection buffer", "err", err)
  425. // return nil
  426. }
  427. c.logger.Info("Peek connection buffer", "numBytes", numBytes, "bz", bz)
  428. }
  429. */
  430. // Read packet type
  431. var packet tmp2p.Packet
  432. _n, err := protoReader.ReadMsg(&packet)
  433. c.recvMonitor.Update(_n)
  434. if err != nil {
  435. // stopServices was invoked and we are shutting down
  436. // receiving is excpected to fail since we will close the connection
  437. select {
  438. case <-ctx.Done():
  439. case <-c.quitRecvRoutine:
  440. break FOR_LOOP
  441. default:
  442. }
  443. if c.IsRunning() {
  444. if err == io.EOF {
  445. c.logger.Info("Connection is closed @ recvRoutine (likely by the other side)", "conn", c)
  446. } else {
  447. c.logger.Debug("Connection failed @ recvRoutine (reading byte)", "conn", c, "err", err)
  448. }
  449. c.stopForError(ctx, err)
  450. }
  451. break FOR_LOOP
  452. }
  453. // Read more depending on packet type.
  454. switch pkt := packet.Sum.(type) {
  455. case *tmp2p.Packet_PacketPing:
  456. // TODO: prevent abuse, as they cause flush()'s.
  457. // https://github.com/tendermint/tendermint/issues/1190
  458. c.logger.Debug("Receive Ping")
  459. select {
  460. case c.pong <- struct{}{}:
  461. default:
  462. // never block
  463. }
  464. case *tmp2p.Packet_PacketPong:
  465. c.logger.Debug("Receive Pong")
  466. select {
  467. case c.pongTimeoutCh <- false:
  468. default:
  469. // never block
  470. }
  471. case *tmp2p.Packet_PacketMsg:
  472. channelID := ChannelID(pkt.PacketMsg.ChannelID)
  473. channel, ok := c.channelsIdx[channelID]
  474. if pkt.PacketMsg.ChannelID < 0 || pkt.PacketMsg.ChannelID > math.MaxUint8 || !ok || channel == nil {
  475. err := fmt.Errorf("unknown channel %X", pkt.PacketMsg.ChannelID)
  476. c.logger.Debug("Connection failed @ recvRoutine", "conn", c, "err", err)
  477. c.stopForError(ctx, err)
  478. break FOR_LOOP
  479. }
  480. msgBytes, err := channel.recvPacketMsg(*pkt.PacketMsg)
  481. if err != nil {
  482. if c.IsRunning() {
  483. c.logger.Debug("Connection failed @ recvRoutine", "conn", c, "err", err)
  484. c.stopForError(ctx, err)
  485. }
  486. break FOR_LOOP
  487. }
  488. if msgBytes != nil {
  489. c.logger.Debug("Received bytes", "chID", channelID, "msgBytes", msgBytes)
  490. // NOTE: This means the reactor.Receive runs in the same thread as the p2p recv routine
  491. c.onReceive(ctx, channelID, msgBytes)
  492. }
  493. default:
  494. err := fmt.Errorf("unknown message type %v", reflect.TypeOf(packet))
  495. c.logger.Error("Connection failed @ recvRoutine", "conn", c, "err", err)
  496. c.stopForError(ctx, err)
  497. break FOR_LOOP
  498. }
  499. }
  500. // Cleanup
  501. close(c.pong)
  502. for range c.pong {
  503. // Drain
  504. }
  505. }
  506. // not goroutine-safe
  507. func (c *MConnection) stopPongTimer() {
  508. if c.pongTimer != nil {
  509. _ = c.pongTimer.Stop()
  510. c.pongTimer = nil
  511. }
  512. }
  513. // maxPacketMsgSize returns a maximum size of PacketMsg
  514. func (c *MConnection) maxPacketMsgSize() int {
  515. bz, err := proto.Marshal(mustWrapPacket(&tmp2p.PacketMsg{
  516. ChannelID: 0x01,
  517. EOF: true,
  518. Data: make([]byte, c.config.MaxPacketMsgPayloadSize),
  519. }))
  520. if err != nil {
  521. panic(err)
  522. }
  523. return len(bz)
  524. }
  525. type ChannelStatus struct {
  526. ID byte
  527. SendQueueCapacity int
  528. SendQueueSize int
  529. Priority int
  530. RecentlySent int64
  531. }
  532. //-----------------------------------------------------------------------------
  533. // ChannelID is an arbitrary channel ID.
  534. type ChannelID uint16
  535. type ChannelDescriptor struct {
  536. ID ChannelID
  537. Priority int
  538. MessageType proto.Message
  539. // TODO: Remove once p2p refactor is complete.
  540. SendQueueCapacity int
  541. RecvMessageCapacity int
  542. // RecvBufferCapacity defines the max buffer size of inbound messages for a
  543. // given p2p Channel queue.
  544. RecvBufferCapacity int
  545. }
  546. func (chDesc ChannelDescriptor) FillDefaults() (filled ChannelDescriptor) {
  547. if chDesc.SendQueueCapacity == 0 {
  548. chDesc.SendQueueCapacity = defaultSendQueueCapacity
  549. }
  550. if chDesc.RecvBufferCapacity == 0 {
  551. chDesc.RecvBufferCapacity = defaultRecvBufferCapacity
  552. }
  553. if chDesc.RecvMessageCapacity == 0 {
  554. chDesc.RecvMessageCapacity = defaultRecvMessageCapacity
  555. }
  556. filled = chDesc
  557. return
  558. }
  559. // NOTE: not goroutine-safe.
  560. type channel struct {
  561. // Exponential moving average.
  562. // This field must be accessed atomically.
  563. // It is first in the struct to ensure correct alignment.
  564. // See https://github.com/tendermint/tendermint/issues/7000.
  565. recentlySent int64
  566. conn *MConnection
  567. desc ChannelDescriptor
  568. sendQueue chan []byte
  569. sendQueueSize int32 // atomic.
  570. recving []byte
  571. sending []byte
  572. maxPacketMsgPayloadSize int
  573. logger log.Logger
  574. }
  575. func newChannel(conn *MConnection, desc ChannelDescriptor) *channel {
  576. desc = desc.FillDefaults()
  577. if desc.Priority <= 0 {
  578. panic("Channel default priority must be a positive integer")
  579. }
  580. return &channel{
  581. conn: conn,
  582. desc: desc,
  583. sendQueue: make(chan []byte, desc.SendQueueCapacity),
  584. recving: make([]byte, 0, desc.RecvBufferCapacity),
  585. maxPacketMsgPayloadSize: conn.config.MaxPacketMsgPayloadSize,
  586. logger: conn.logger,
  587. }
  588. }
  589. // Queues message to send to this channel.
  590. // Goroutine-safe
  591. // Times out (and returns false) after defaultSendTimeout
  592. func (ch *channel) sendBytes(bytes []byte) bool {
  593. select {
  594. case ch.sendQueue <- bytes:
  595. atomic.AddInt32(&ch.sendQueueSize, 1)
  596. return true
  597. case <-time.After(defaultSendTimeout):
  598. return false
  599. }
  600. }
  601. // Returns true if any PacketMsgs are pending to be sent.
  602. // Call before calling nextPacketMsg()
  603. // Goroutine-safe
  604. func (ch *channel) isSendPending() bool {
  605. if len(ch.sending) == 0 {
  606. if len(ch.sendQueue) == 0 {
  607. return false
  608. }
  609. ch.sending = <-ch.sendQueue
  610. }
  611. return true
  612. }
  613. // Creates a new PacketMsg to send.
  614. // Not goroutine-safe
  615. func (ch *channel) nextPacketMsg() tmp2p.PacketMsg {
  616. packet := tmp2p.PacketMsg{ChannelID: int32(ch.desc.ID)}
  617. maxSize := ch.maxPacketMsgPayloadSize
  618. packet.Data = ch.sending[:tmmath.MinInt(maxSize, len(ch.sending))]
  619. if len(ch.sending) <= maxSize {
  620. packet.EOF = true
  621. ch.sending = nil
  622. atomic.AddInt32(&ch.sendQueueSize, -1) // decrement sendQueueSize
  623. } else {
  624. packet.EOF = false
  625. ch.sending = ch.sending[tmmath.MinInt(maxSize, len(ch.sending)):]
  626. }
  627. return packet
  628. }
  629. // Writes next PacketMsg to w and updates c.recentlySent.
  630. // Not goroutine-safe
  631. func (ch *channel) writePacketMsgTo(w io.Writer) (n int, err error) {
  632. packet := ch.nextPacketMsg()
  633. n, err = protoio.NewDelimitedWriter(w).WriteMsg(mustWrapPacket(&packet))
  634. atomic.AddInt64(&ch.recentlySent, int64(n))
  635. return
  636. }
  637. // Handles incoming PacketMsgs. It returns a message bytes if message is
  638. // complete, which is owned by the caller and will not be modified.
  639. // Not goroutine-safe
  640. func (ch *channel) recvPacketMsg(packet tmp2p.PacketMsg) ([]byte, error) {
  641. ch.logger.Debug("Read PacketMsg", "conn", ch.conn, "packet", packet)
  642. var recvCap, recvReceived = ch.desc.RecvMessageCapacity, len(ch.recving) + len(packet.Data)
  643. if recvCap < recvReceived {
  644. return nil, fmt.Errorf("received message exceeds available capacity: %v < %v", recvCap, recvReceived)
  645. }
  646. ch.recving = append(ch.recving, packet.Data...)
  647. if packet.EOF {
  648. msgBytes := ch.recving
  649. ch.recving = make([]byte, 0, ch.desc.RecvBufferCapacity)
  650. return msgBytes, nil
  651. }
  652. return nil, nil
  653. }
  654. // Call this periodically to update stats for throttling purposes.
  655. // Not goroutine-safe
  656. func (ch *channel) updateStats() {
  657. // Exponential decay of stats.
  658. // TODO: optimize.
  659. atomic.StoreInt64(&ch.recentlySent, int64(float64(atomic.LoadInt64(&ch.recentlySent))*0.8))
  660. }
  661. //----------------------------------------
  662. // Packet
  663. // mustWrapPacket takes a packet kind (oneof) and wraps it in a tmp2p.Packet message.
  664. func mustWrapPacket(pb proto.Message) *tmp2p.Packet {
  665. var msg tmp2p.Packet
  666. switch pb := pb.(type) {
  667. case *tmp2p.Packet: // already a packet
  668. msg = *pb
  669. case *tmp2p.PacketPing:
  670. msg = tmp2p.Packet{
  671. Sum: &tmp2p.Packet_PacketPing{
  672. PacketPing: pb,
  673. },
  674. }
  675. case *tmp2p.PacketPong:
  676. msg = tmp2p.Packet{
  677. Sum: &tmp2p.Packet_PacketPong{
  678. PacketPong: pb,
  679. },
  680. }
  681. case *tmp2p.PacketMsg:
  682. msg = tmp2p.Packet{
  683. Sum: &tmp2p.Packet_PacketMsg{
  684. PacketMsg: pb,
  685. },
  686. }
  687. default:
  688. panic(fmt.Errorf("unknown packet type %T", pb))
  689. }
  690. return &msg
  691. }