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.

483 lines
13 KiB

7 years ago
7 years ago
7 years ago
7 years ago
new pubsub package comment out failing consensus tests for now rewrite rpc httpclient to use new pubsub package import pubsub as tmpubsub, query as tmquery make event IDs constants EventKey -> EventTypeKey rename EventsPubsub to PubSub mempool does not use pubsub rename eventsSub to pubsub new subscribe API fix channel size issues and consensus tests bugs refactor rpc client add missing discardFromChan method add mutex rename pubsub to eventBus remove IsRunning from WSRPCConnection interface (not needed) add a comment in broadcastNewRoundStepsAndVotes rename registerEventCallbacks to broadcastNewRoundStepsAndVotes See https://dave.cheney.net/2014/03/19/channel-axioms stop eventBuses after reactor tests remove unnecessary Unsubscribe return subscribe helper function move discardFromChan to where it is used subscribe now returns an err this gives us ability to refuse to subscribe if pubsub is at its max capacity. use context for control overflow cache queries handle err when subscribing in replay_test rename testClientID to testSubscriber extract var set channel buffer capacity to 1 in replay_file fix byzantine_test unsubscribe from single event, not all events refactor httpclient to return events to appropriate channels return failing testReplayCrashBeforeWriteVote test fix TestValidatorSetChanges refactor code a bit fix testReplayCrashBeforeWriteVote add comment fix TestValidatorSetChanges fixes from Bucky's review update comment [ci skip] test TxEventBuffer update changelog fix TestValidatorSetChanges (2nd attempt) only do wg.Done when no errors benchmark event bus create pubsub server inside NewEventBus only expose config params (later if needed) set buffer capacity to 0 so we are not testing cache new tx event format: key = "Tx" plus a tag {"tx.hash": XYZ} This should allow to subscribe to all transactions! or a specific one using a query: "tm.events.type = Tx and tx.hash = '013ABF99434...'" use TimeoutCommit instead of afterPublishEventNewBlockTimeout TimeoutCommit is the time a node waits after committing a block, before it goes into the next height. So it will finish everything from the last block, but then wait a bit. The idea is this gives it time to hear more votes from other validators, to strengthen the commit it includes in the next block. But it also gives it time to hear about new transactions. waitForBlockWithUpdatedVals rewrite WAL crash tests Task: test that we can recover from any WAL crash. Solution: the old tests were relying on event hub being run in the same thread (we were injecting the private validator's last signature). when considering a rewrite, we considered two possible solutions: write a "fuzzy" testing system where WAL is crashing upon receiving a new message, or inject failures and trigger them in tests using something like https://github.com/coreos/gofail. remove sleep no cs.Lock around wal.Save test different cases (empty block, non-empty block, ...) comments add comments test 4 cases: empty block, non-empty block, non-empty block with smaller part size, many blocks fixes as per Bucky's last review reset subscriptions on UnsubscribeAll use a simple counter to track message for which we panicked also, set a smaller part size for all test cases
7 years ago
new pubsub package comment out failing consensus tests for now rewrite rpc httpclient to use new pubsub package import pubsub as tmpubsub, query as tmquery make event IDs constants EventKey -> EventTypeKey rename EventsPubsub to PubSub mempool does not use pubsub rename eventsSub to pubsub new subscribe API fix channel size issues and consensus tests bugs refactor rpc client add missing discardFromChan method add mutex rename pubsub to eventBus remove IsRunning from WSRPCConnection interface (not needed) add a comment in broadcastNewRoundStepsAndVotes rename registerEventCallbacks to broadcastNewRoundStepsAndVotes See https://dave.cheney.net/2014/03/19/channel-axioms stop eventBuses after reactor tests remove unnecessary Unsubscribe return subscribe helper function move discardFromChan to where it is used subscribe now returns an err this gives us ability to refuse to subscribe if pubsub is at its max capacity. use context for control overflow cache queries handle err when subscribing in replay_test rename testClientID to testSubscriber extract var set channel buffer capacity to 1 in replay_file fix byzantine_test unsubscribe from single event, not all events refactor httpclient to return events to appropriate channels return failing testReplayCrashBeforeWriteVote test fix TestValidatorSetChanges refactor code a bit fix testReplayCrashBeforeWriteVote add comment fix TestValidatorSetChanges fixes from Bucky's review update comment [ci skip] test TxEventBuffer update changelog fix TestValidatorSetChanges (2nd attempt) only do wg.Done when no errors benchmark event bus create pubsub server inside NewEventBus only expose config params (later if needed) set buffer capacity to 0 so we are not testing cache new tx event format: key = "Tx" plus a tag {"tx.hash": XYZ} This should allow to subscribe to all transactions! or a specific one using a query: "tm.events.type = Tx and tx.hash = '013ABF99434...'" use TimeoutCommit instead of afterPublishEventNewBlockTimeout TimeoutCommit is the time a node waits after committing a block, before it goes into the next height. So it will finish everything from the last block, but then wait a bit. The idea is this gives it time to hear more votes from other validators, to strengthen the commit it includes in the next block. But it also gives it time to hear about new transactions. waitForBlockWithUpdatedVals rewrite WAL crash tests Task: test that we can recover from any WAL crash. Solution: the old tests were relying on event hub being run in the same thread (we were injecting the private validator's last signature). when considering a rewrite, we considered two possible solutions: write a "fuzzy" testing system where WAL is crashing upon receiving a new message, or inject failures and trigger them in tests using something like https://github.com/coreos/gofail. remove sleep no cs.Lock around wal.Save test different cases (empty block, non-empty block, ...) comments add comments test 4 cases: empty block, non-empty block, non-empty block with smaller part size, many blocks fixes as per Bucky's last review reset subscriptions on UnsubscribeAll use a simple counter to track message for which we panicked also, set a smaller part size for all test cases
7 years ago
new pubsub package comment out failing consensus tests for now rewrite rpc httpclient to use new pubsub package import pubsub as tmpubsub, query as tmquery make event IDs constants EventKey -> EventTypeKey rename EventsPubsub to PubSub mempool does not use pubsub rename eventsSub to pubsub new subscribe API fix channel size issues and consensus tests bugs refactor rpc client add missing discardFromChan method add mutex rename pubsub to eventBus remove IsRunning from WSRPCConnection interface (not needed) add a comment in broadcastNewRoundStepsAndVotes rename registerEventCallbacks to broadcastNewRoundStepsAndVotes See https://dave.cheney.net/2014/03/19/channel-axioms stop eventBuses after reactor tests remove unnecessary Unsubscribe return subscribe helper function move discardFromChan to where it is used subscribe now returns an err this gives us ability to refuse to subscribe if pubsub is at its max capacity. use context for control overflow cache queries handle err when subscribing in replay_test rename testClientID to testSubscriber extract var set channel buffer capacity to 1 in replay_file fix byzantine_test unsubscribe from single event, not all events refactor httpclient to return events to appropriate channels return failing testReplayCrashBeforeWriteVote test fix TestValidatorSetChanges refactor code a bit fix testReplayCrashBeforeWriteVote add comment fix TestValidatorSetChanges fixes from Bucky's review update comment [ci skip] test TxEventBuffer update changelog fix TestValidatorSetChanges (2nd attempt) only do wg.Done when no errors benchmark event bus create pubsub server inside NewEventBus only expose config params (later if needed) set buffer capacity to 0 so we are not testing cache new tx event format: key = "Tx" plus a tag {"tx.hash": XYZ} This should allow to subscribe to all transactions! or a specific one using a query: "tm.events.type = Tx and tx.hash = '013ABF99434...'" use TimeoutCommit instead of afterPublishEventNewBlockTimeout TimeoutCommit is the time a node waits after committing a block, before it goes into the next height. So it will finish everything from the last block, but then wait a bit. The idea is this gives it time to hear more votes from other validators, to strengthen the commit it includes in the next block. But it also gives it time to hear about new transactions. waitForBlockWithUpdatedVals rewrite WAL crash tests Task: test that we can recover from any WAL crash. Solution: the old tests were relying on event hub being run in the same thread (we were injecting the private validator's last signature). when considering a rewrite, we considered two possible solutions: write a "fuzzy" testing system where WAL is crashing upon receiving a new message, or inject failures and trigger them in tests using something like https://github.com/coreos/gofail. remove sleep no cs.Lock around wal.Save test different cases (empty block, non-empty block, ...) comments add comments test 4 cases: empty block, non-empty block, non-empty block with smaller part size, many blocks fixes as per Bucky's last review reset subscriptions on UnsubscribeAll use a simple counter to track message for which we panicked also, set a smaller part size for all test cases
7 years ago
new pubsub package comment out failing consensus tests for now rewrite rpc httpclient to use new pubsub package import pubsub as tmpubsub, query as tmquery make event IDs constants EventKey -> EventTypeKey rename EventsPubsub to PubSub mempool does not use pubsub rename eventsSub to pubsub new subscribe API fix channel size issues and consensus tests bugs refactor rpc client add missing discardFromChan method add mutex rename pubsub to eventBus remove IsRunning from WSRPCConnection interface (not needed) add a comment in broadcastNewRoundStepsAndVotes rename registerEventCallbacks to broadcastNewRoundStepsAndVotes See https://dave.cheney.net/2014/03/19/channel-axioms stop eventBuses after reactor tests remove unnecessary Unsubscribe return subscribe helper function move discardFromChan to where it is used subscribe now returns an err this gives us ability to refuse to subscribe if pubsub is at its max capacity. use context for control overflow cache queries handle err when subscribing in replay_test rename testClientID to testSubscriber extract var set channel buffer capacity to 1 in replay_file fix byzantine_test unsubscribe from single event, not all events refactor httpclient to return events to appropriate channels return failing testReplayCrashBeforeWriteVote test fix TestValidatorSetChanges refactor code a bit fix testReplayCrashBeforeWriteVote add comment fix TestValidatorSetChanges fixes from Bucky's review update comment [ci skip] test TxEventBuffer update changelog fix TestValidatorSetChanges (2nd attempt) only do wg.Done when no errors benchmark event bus create pubsub server inside NewEventBus only expose config params (later if needed) set buffer capacity to 0 so we are not testing cache new tx event format: key = "Tx" plus a tag {"tx.hash": XYZ} This should allow to subscribe to all transactions! or a specific one using a query: "tm.events.type = Tx and tx.hash = '013ABF99434...'" use TimeoutCommit instead of afterPublishEventNewBlockTimeout TimeoutCommit is the time a node waits after committing a block, before it goes into the next height. So it will finish everything from the last block, but then wait a bit. The idea is this gives it time to hear more votes from other validators, to strengthen the commit it includes in the next block. But it also gives it time to hear about new transactions. waitForBlockWithUpdatedVals rewrite WAL crash tests Task: test that we can recover from any WAL crash. Solution: the old tests were relying on event hub being run in the same thread (we were injecting the private validator's last signature). when considering a rewrite, we considered two possible solutions: write a "fuzzy" testing system where WAL is crashing upon receiving a new message, or inject failures and trigger them in tests using something like https://github.com/coreos/gofail. remove sleep no cs.Lock around wal.Save test different cases (empty block, non-empty block, ...) comments add comments test 4 cases: empty block, non-empty block, non-empty block with smaller part size, many blocks fixes as per Bucky's last review reset subscriptions on UnsubscribeAll use a simple counter to track message for which we panicked also, set a smaller part size for all test cases
7 years ago
  1. package rpcclient
  2. import (
  3. "context"
  4. "encoding/json"
  5. "fmt"
  6. "math/rand"
  7. "net"
  8. "net/http"
  9. "sync"
  10. "time"
  11. "github.com/gorilla/websocket"
  12. "github.com/pkg/errors"
  13. metrics "github.com/rcrowley/go-metrics"
  14. types "github.com/tendermint/tendermint/rpc/lib/types"
  15. cmn "github.com/tendermint/tmlibs/common"
  16. )
  17. const (
  18. defaultMaxReconnectAttempts = 25
  19. defaultWriteWait = 0
  20. defaultReadWait = 0
  21. defaultPingPeriod = 0
  22. )
  23. // WSClient is a WebSocket client. The methods of WSClient are safe for use by
  24. // multiple goroutines.
  25. type WSClient struct {
  26. cmn.BaseService
  27. conn *websocket.Conn
  28. Address string // IP:PORT or /path/to/socket
  29. Endpoint string // /websocket/url/endpoint
  30. Dialer func(string, string) (net.Conn, error)
  31. // Time between sending a ping and receiving a pong. See
  32. // https://godoc.org/github.com/rcrowley/go-metrics#Timer.
  33. PingPongLatencyTimer metrics.Timer
  34. // Single user facing channel to read RPCResponses from, closed only when the client is being stopped.
  35. ResponsesCh chan types.RPCResponse
  36. // Callback, which will be called each time after successful reconnect.
  37. onReconnect func()
  38. // internal channels
  39. send chan types.RPCRequest // user requests
  40. backlog chan types.RPCRequest // stores a single user request received during a conn failure
  41. reconnectAfter chan error // reconnect requests
  42. readRoutineQuit chan struct{} // a way for readRoutine to close writeRoutine
  43. wg sync.WaitGroup
  44. mtx sync.RWMutex
  45. sentLastPingAt time.Time
  46. reconnecting bool
  47. // Maximum reconnect attempts (0 or greater; default: 25).
  48. maxReconnectAttempts int
  49. // Time allowed to write a message to the server. 0 means block until operation succeeds.
  50. writeWait time.Duration
  51. // Time allowed to read the next message from the server. 0 means block until operation succeeds.
  52. readWait time.Duration
  53. // Send pings to server with this period. Must be less than readWait. If 0, no pings will be sent.
  54. pingPeriod time.Duration
  55. }
  56. // NewWSClient returns a new client. See the commentary on the func(*WSClient)
  57. // functions for a detailed description of how to configure ping period and
  58. // pong wait time. The endpoint argument must begin with a `/`.
  59. func NewWSClient(remoteAddr, endpoint string, options ...func(*WSClient)) *WSClient {
  60. addr, dialer := makeHTTPDialer(remoteAddr)
  61. c := &WSClient{
  62. Address: addr,
  63. Dialer: dialer,
  64. Endpoint: endpoint,
  65. PingPongLatencyTimer: metrics.NewTimer(),
  66. maxReconnectAttempts: defaultMaxReconnectAttempts,
  67. readWait: defaultReadWait,
  68. writeWait: defaultWriteWait,
  69. pingPeriod: defaultPingPeriod,
  70. }
  71. c.BaseService = *cmn.NewBaseService(nil, "WSClient", c)
  72. for _, option := range options {
  73. option(c)
  74. }
  75. return c
  76. }
  77. // MaxReconnectAttempts sets the maximum number of reconnect attempts before returning an error.
  78. // It should only be used in the constructor and is not Goroutine-safe.
  79. func MaxReconnectAttempts(max int) func(*WSClient) {
  80. return func(c *WSClient) {
  81. c.maxReconnectAttempts = max
  82. }
  83. }
  84. // ReadWait sets the amount of time to wait before a websocket read times out.
  85. // It should only be used in the constructor and is not Goroutine-safe.
  86. func ReadWait(readWait time.Duration) func(*WSClient) {
  87. return func(c *WSClient) {
  88. c.readWait = readWait
  89. }
  90. }
  91. // WriteWait sets the amount of time to wait before a websocket write times out.
  92. // It should only be used in the constructor and is not Goroutine-safe.
  93. func WriteWait(writeWait time.Duration) func(*WSClient) {
  94. return func(c *WSClient) {
  95. c.writeWait = writeWait
  96. }
  97. }
  98. // PingPeriod sets the duration for sending websocket pings.
  99. // It should only be used in the constructor - not Goroutine-safe.
  100. func PingPeriod(pingPeriod time.Duration) func(*WSClient) {
  101. return func(c *WSClient) {
  102. c.pingPeriod = pingPeriod
  103. }
  104. }
  105. // OnReconnect sets the callback, which will be called every time after
  106. // successful reconnect.
  107. func OnReconnect(cb func()) func(*WSClient) {
  108. return func(c *WSClient) {
  109. c.onReconnect = cb
  110. }
  111. }
  112. // String returns WS client full address.
  113. func (c *WSClient) String() string {
  114. return fmt.Sprintf("%s (%s)", c.Address, c.Endpoint)
  115. }
  116. // OnStart implements cmn.Service by dialing a server and creating read and
  117. // write routines.
  118. func (c *WSClient) OnStart() error {
  119. err := c.dial()
  120. if err != nil {
  121. return err
  122. }
  123. c.ResponsesCh = make(chan types.RPCResponse)
  124. c.send = make(chan types.RPCRequest)
  125. // 1 additional error may come from the read/write
  126. // goroutine depending on which failed first.
  127. c.reconnectAfter = make(chan error, 1)
  128. // capacity for 1 request. a user won't be able to send more because the send
  129. // channel is unbuffered.
  130. c.backlog = make(chan types.RPCRequest, 1)
  131. c.startReadWriteRoutines()
  132. go c.reconnectRoutine()
  133. return nil
  134. }
  135. // OnStop implements cmn.Service.
  136. func (c *WSClient) OnStop() {}
  137. // Stop overrides cmn.Service#Stop. There is no other way to wait until Quit
  138. // channel is closed.
  139. func (c *WSClient) Stop() bool {
  140. success := c.BaseService.Stop()
  141. // only close user-facing channels when we can't write to them
  142. c.wg.Wait()
  143. close(c.ResponsesCh)
  144. return success
  145. }
  146. // IsReconnecting returns true if the client is reconnecting right now.
  147. func (c *WSClient) IsReconnecting() bool {
  148. c.mtx.RLock()
  149. defer c.mtx.RUnlock()
  150. return c.reconnecting
  151. }
  152. // IsActive returns true if the client is running and not reconnecting.
  153. func (c *WSClient) IsActive() bool {
  154. return c.IsRunning() && !c.IsReconnecting()
  155. }
  156. // Send the given RPC request to the server. Results will be available on
  157. // ResponsesCh, errors, if any, on ErrorsCh. Will block until send succeeds or
  158. // ctx.Done is closed.
  159. func (c *WSClient) Send(ctx context.Context, request types.RPCRequest) error {
  160. select {
  161. case c.send <- request:
  162. c.Logger.Info("sent a request", "req", request)
  163. return nil
  164. case <-ctx.Done():
  165. return ctx.Err()
  166. }
  167. }
  168. // Call the given method. See Send description.
  169. func (c *WSClient) Call(ctx context.Context, method string, params map[string]interface{}) error {
  170. request, err := types.MapToRequest("ws-client", method, params)
  171. if err != nil {
  172. return err
  173. }
  174. return c.Send(ctx, request)
  175. }
  176. // CallWithArrayParams the given method with params in a form of array. See
  177. // Send description.
  178. func (c *WSClient) CallWithArrayParams(ctx context.Context, method string, params []interface{}) error {
  179. request, err := types.ArrayToRequest("ws-client", method, params)
  180. if err != nil {
  181. return err
  182. }
  183. return c.Send(ctx, request)
  184. }
  185. ///////////////////////////////////////////////////////////////////////////////
  186. // Private methods
  187. func (c *WSClient) dial() error {
  188. dialer := &websocket.Dialer{
  189. NetDial: c.Dialer,
  190. Proxy: http.ProxyFromEnvironment,
  191. }
  192. rHeader := http.Header{}
  193. conn, _, err := dialer.Dial("ws://"+c.Address+c.Endpoint, rHeader)
  194. if err != nil {
  195. return err
  196. }
  197. c.conn = conn
  198. return nil
  199. }
  200. // reconnect tries to redial up to maxReconnectAttempts with exponential
  201. // backoff.
  202. func (c *WSClient) reconnect() error {
  203. attempt := 0
  204. c.mtx.Lock()
  205. c.reconnecting = true
  206. c.mtx.Unlock()
  207. defer func() {
  208. c.mtx.Lock()
  209. c.reconnecting = false
  210. c.mtx.Unlock()
  211. }()
  212. // 1s == (1e9 ns) == (1 Billion ns)
  213. billionNs := float64(time.Second.Nanoseconds())
  214. for {
  215. jitterSeconds := time.Duration(rand.Float64() * billionNs)
  216. backoffDuration := jitterSeconds + ((1 << uint(attempt)) * time.Second)
  217. c.Logger.Info("reconnecting", "attempt", attempt+1, "backoff_duration", backoffDuration)
  218. time.Sleep(backoffDuration)
  219. err := c.dial()
  220. if err != nil {
  221. c.Logger.Error("failed to redial", "err", err)
  222. } else {
  223. c.Logger.Info("reconnected")
  224. if c.onReconnect != nil {
  225. go c.onReconnect()
  226. }
  227. return nil
  228. }
  229. attempt++
  230. if attempt > c.maxReconnectAttempts {
  231. return errors.Wrap(err, "reached maximum reconnect attempts")
  232. }
  233. }
  234. }
  235. func (c *WSClient) startReadWriteRoutines() {
  236. c.wg.Add(2)
  237. c.readRoutineQuit = make(chan struct{})
  238. go c.readRoutine()
  239. go c.writeRoutine()
  240. }
  241. func (c *WSClient) processBacklog() error {
  242. select {
  243. case request := <-c.backlog:
  244. if c.writeWait > 0 {
  245. if err := c.conn.SetWriteDeadline(time.Now().Add(c.writeWait)); err != nil {
  246. c.Logger.Error("failed to set write deadline", "err", err)
  247. }
  248. }
  249. if err := c.conn.WriteJSON(request); err != nil {
  250. c.Logger.Error("failed to resend request", "err", err)
  251. c.reconnectAfter <- err
  252. // requeue request
  253. c.backlog <- request
  254. return err
  255. }
  256. c.Logger.Info("resend a request", "req", request)
  257. default:
  258. }
  259. return nil
  260. }
  261. func (c *WSClient) reconnectRoutine() {
  262. for {
  263. select {
  264. case originalError := <-c.reconnectAfter:
  265. // wait until writeRoutine and readRoutine finish
  266. c.wg.Wait()
  267. if err := c.reconnect(); err != nil {
  268. c.Logger.Error("failed to reconnect", "err", err, "original_err", originalError)
  269. c.Stop()
  270. return
  271. } else {
  272. // drain reconnectAfter
  273. LOOP:
  274. for {
  275. select {
  276. case <-c.reconnectAfter:
  277. default:
  278. break LOOP
  279. }
  280. }
  281. err = c.processBacklog()
  282. if err == nil {
  283. c.startReadWriteRoutines()
  284. }
  285. }
  286. case <-c.Quit:
  287. return
  288. }
  289. }
  290. }
  291. // The client ensures that there is at most one writer to a connection by
  292. // executing all writes from this goroutine.
  293. func (c *WSClient) writeRoutine() {
  294. var ticker *time.Ticker
  295. if c.pingPeriod > 0 {
  296. // ticker with a predefined period
  297. ticker = time.NewTicker(c.pingPeriod)
  298. } else {
  299. // ticker that never fires
  300. ticker = &time.Ticker{C: make(<-chan time.Time)}
  301. }
  302. defer func() {
  303. ticker.Stop()
  304. if err := c.conn.Close(); err != nil {
  305. // ignore error; it will trigger in tests
  306. // likely because it's closing and already closed connection
  307. }
  308. c.wg.Done()
  309. }()
  310. for {
  311. select {
  312. case request := <-c.send:
  313. if c.writeWait > 0 {
  314. if err := c.conn.SetWriteDeadline(time.Now().Add(c.writeWait)); err != nil {
  315. c.Logger.Error("failed to set write deadline", "err", err)
  316. }
  317. }
  318. if err := c.conn.WriteJSON(request); err != nil {
  319. c.Logger.Error("failed to send request", "err", err)
  320. c.reconnectAfter <- err
  321. // add request to the backlog, so we don't lose it
  322. c.backlog <- request
  323. return
  324. }
  325. case <-ticker.C:
  326. if c.writeWait > 0 {
  327. if err := c.conn.SetWriteDeadline(time.Now().Add(c.writeWait)); err != nil {
  328. c.Logger.Error("failed to set write deadline", "err", err)
  329. }
  330. }
  331. if err := c.conn.WriteMessage(websocket.PingMessage, []byte{}); err != nil {
  332. c.Logger.Error("failed to write ping", "err", err)
  333. c.reconnectAfter <- err
  334. return
  335. }
  336. c.mtx.Lock()
  337. c.sentLastPingAt = time.Now()
  338. c.mtx.Unlock()
  339. c.Logger.Debug("sent ping")
  340. case <-c.readRoutineQuit:
  341. return
  342. case <-c.Quit:
  343. if err := c.conn.WriteMessage(websocket.CloseMessage, websocket.FormatCloseMessage(websocket.CloseNormalClosure, "")); err != nil {
  344. c.Logger.Error("failed to write message", "err", err)
  345. }
  346. return
  347. }
  348. }
  349. }
  350. // The client ensures that there is at most one reader to a connection by
  351. // executing all reads from this goroutine.
  352. func (c *WSClient) readRoutine() {
  353. defer func() {
  354. if err := c.conn.Close(); err != nil {
  355. // ignore error; it will trigger in tests
  356. // likely because it's closing and already closed connection
  357. }
  358. c.wg.Done()
  359. }()
  360. c.conn.SetPongHandler(func(string) error {
  361. // gather latency stats
  362. c.mtx.RLock()
  363. t := c.sentLastPingAt
  364. c.mtx.RUnlock()
  365. c.PingPongLatencyTimer.UpdateSince(t)
  366. c.Logger.Debug("got pong")
  367. return nil
  368. })
  369. for {
  370. // reset deadline for every message type (control or data)
  371. if c.readWait > 0 {
  372. if err := c.conn.SetReadDeadline(time.Now().Add(c.readWait)); err != nil {
  373. c.Logger.Error("failed to set read deadline", "err", err)
  374. }
  375. }
  376. _, data, err := c.conn.ReadMessage()
  377. if err != nil {
  378. if !websocket.IsUnexpectedCloseError(err, websocket.CloseNormalClosure) {
  379. return
  380. }
  381. c.Logger.Error("failed to read response", "err", err)
  382. close(c.readRoutineQuit)
  383. c.reconnectAfter <- err
  384. return
  385. }
  386. var response types.RPCResponse
  387. err = json.Unmarshal(data, &response)
  388. if err != nil {
  389. c.Logger.Error("failed to parse response", "err", err, "data", string(data))
  390. continue
  391. }
  392. c.Logger.Info("got response", "resp", response.Result)
  393. // Combine a non-blocking read on BaseService.Quit with a non-blocking write on ResponsesCh to avoid blocking
  394. // c.wg.Wait() in c.Stop(). Note we rely on Quit being closed so that it sends unlimited Quit signals to stop
  395. // both readRoutine and writeRoutine
  396. select {
  397. case <-c.Quit:
  398. case c.ResponsesCh <- response:
  399. }
  400. }
  401. }
  402. ///////////////////////////////////////////////////////////////////////////////
  403. // Predefined methods
  404. // Subscribe to a query. Note the server must have a "subscribe" route
  405. // defined.
  406. func (c *WSClient) Subscribe(ctx context.Context, query string) error {
  407. params := map[string]interface{}{"query": query}
  408. return c.Call(ctx, "subscribe", params)
  409. }
  410. // Unsubscribe from a query. Note the server must have a "unsubscribe" route
  411. // defined.
  412. func (c *WSClient) Unsubscribe(ctx context.Context, query string) error {
  413. params := map[string]interface{}{"query": query}
  414. return c.Call(ctx, "unsubscribe", params)
  415. }
  416. // UnsubscribeAll from all. Note the server must have a "unsubscribe_all" route
  417. // defined.
  418. func (c *WSClient) UnsubscribeAll(ctx context.Context) error {
  419. params := map[string]interface{}{}
  420. return c.Call(ctx, "unsubscribe_all", params)
  421. }