consumer.go 19 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647
  1. package sarama
  2. import (
  3. "fmt"
  4. "sync"
  5. "sync/atomic"
  6. "time"
  7. )
  8. // ConsumerMessage encapsulates a Kafka message returned by the consumer.
  9. type ConsumerMessage struct {
  10. Key, Value []byte
  11. Topic string
  12. Partition int32
  13. Offset int64
  14. }
  15. // ConsumerError is what is provided to the user when an error occurs.
  16. // It wraps an error and includes the topic and partition.
  17. type ConsumerError struct {
  18. Topic string
  19. Partition int32
  20. Err error
  21. }
  22. func (ce ConsumerError) Error() string {
  23. return fmt.Sprintf("kafka: error while consuming %s/%d: %s", ce.Topic, ce.Partition, ce.Err)
  24. }
  25. // ConsumerErrors is a type that wraps a batch of errors and implements the Error interface.
  26. // It can be returned from the PartitionConsumer's Close methods to avoid the need to manually drain errors
  27. // when stopping.
  28. type ConsumerErrors []*ConsumerError
  29. func (ce ConsumerErrors) Error() string {
  30. return fmt.Sprintf("kafka: %d errors while consuming", len(ce))
  31. }
  32. // Consumer manages PartitionConsumers which process Kafka messages from brokers. You MUST call Close()
  33. // on a consumer to avoid leaks, it will not be garbage-collected automatically when it passes out of
  34. // scope.
  35. //
  36. // Sarama's Consumer type does not currently support automatic consumer group rebalancing and offset tracking,
  37. // however the https://github.com/wvanbergen/kafka library builds on Sarama to add this support. We plan
  38. // to properly integrate this functionality at a later date.
  39. type Consumer interface {
  40. // Topics returns the set of available topics as retrieved from the cluster metadata.
  41. // This method is the same as Client.Topics(), and is provided for convenience.
  42. Topics() ([]string, error)
  43. // Partitions returns the sorted list of all partition IDs for the given topic.
  44. // This method is the same as Client.Pertitions(), and is provided for convenience.
  45. Partitions(topic string) ([]int32, error)
  46. // ConsumePartition creates a PartitionConsumer on the given topic/partition with the given offset. It will
  47. // return an error if this Consumer is already consuming on the given topic/partition. Offset can be a
  48. // literal offset, or OffsetNewest or OffsetOldest
  49. ConsumePartition(topic string, partition int32, offset int64) (PartitionConsumer, error)
  50. // Close shuts down the consumer. It must be called after all child PartitionConsumers have already been closed.
  51. Close() error
  52. }
  53. type consumer struct {
  54. client Client
  55. conf *Config
  56. ownClient bool
  57. lock sync.Mutex
  58. children map[string]map[int32]*partitionConsumer
  59. brokerConsumers map[*Broker]*brokerConsumer
  60. }
  61. // NewConsumer creates a new consumer using the given broker addresses and configuration.
  62. func NewConsumer(addrs []string, config *Config) (Consumer, error) {
  63. client, err := NewClient(addrs, config)
  64. if err != nil {
  65. return nil, err
  66. }
  67. c, err := NewConsumerFromClient(client)
  68. if err != nil {
  69. return nil, err
  70. }
  71. c.(*consumer).ownClient = true
  72. return c, nil
  73. }
  74. // NewConsumerFromClient creates a new consumer using the given client. It is still
  75. // necessary to call Close() on the underlying client when shutting down this consumer.
  76. func NewConsumerFromClient(client Client) (Consumer, error) {
  77. // Check that we are not dealing with a closed Client before processing any other arguments
  78. if client.Closed() {
  79. return nil, ErrClosedClient
  80. }
  81. c := &consumer{
  82. client: client,
  83. conf: client.Config(),
  84. children: make(map[string]map[int32]*partitionConsumer),
  85. brokerConsumers: make(map[*Broker]*brokerConsumer),
  86. }
  87. return c, nil
  88. }
  89. func (c *consumer) Close() error {
  90. if c.ownClient {
  91. return c.client.Close()
  92. }
  93. return nil
  94. }
  95. func (c *consumer) Topics() ([]string, error) {
  96. return c.client.Topics()
  97. }
  98. func (c *consumer) Partitions(topic string) ([]int32, error) {
  99. return c.client.Partitions(topic)
  100. }
  101. func (c *consumer) ConsumePartition(topic string, partition int32, offset int64) (PartitionConsumer, error) {
  102. child := &partitionConsumer{
  103. consumer: c,
  104. conf: c.conf,
  105. topic: topic,
  106. partition: partition,
  107. messages: make(chan *ConsumerMessage, c.conf.ChannelBufferSize),
  108. errors: make(chan *ConsumerError, c.conf.ChannelBufferSize),
  109. feeder: make(chan *FetchResponse, 1),
  110. trigger: make(chan none, 1),
  111. dying: make(chan error, 1),
  112. fetchSize: c.conf.Consumer.Fetch.Default,
  113. }
  114. if err := child.chooseStartingOffset(offset); err != nil {
  115. return nil, err
  116. }
  117. var leader *Broker
  118. var err error
  119. if leader, err = c.client.Leader(child.topic, child.partition); err != nil {
  120. return nil, err
  121. }
  122. if err := c.addChild(child); err != nil {
  123. return nil, err
  124. }
  125. go withRecover(child.dispatcher)
  126. go withRecover(child.responseFeeder)
  127. child.broker = c.refBrokerConsumer(leader)
  128. child.broker.input <- child
  129. return child, nil
  130. }
  131. func (c *consumer) addChild(child *partitionConsumer) error {
  132. c.lock.Lock()
  133. defer c.lock.Unlock()
  134. topicChildren := c.children[child.topic]
  135. if topicChildren == nil {
  136. topicChildren = make(map[int32]*partitionConsumer)
  137. c.children[child.topic] = topicChildren
  138. }
  139. if topicChildren[child.partition] != nil {
  140. return ConfigurationError("That topic/partition is already being consumed")
  141. }
  142. topicChildren[child.partition] = child
  143. return nil
  144. }
  145. func (c *consumer) removeChild(child *partitionConsumer) {
  146. c.lock.Lock()
  147. defer c.lock.Unlock()
  148. delete(c.children[child.topic], child.partition)
  149. }
  150. func (c *consumer) refBrokerConsumer(broker *Broker) *brokerConsumer {
  151. c.lock.Lock()
  152. defer c.lock.Unlock()
  153. bc := c.brokerConsumers[broker]
  154. if bc == nil {
  155. bc = c.newBrokerConsumer(broker)
  156. c.brokerConsumers[broker] = bc
  157. }
  158. bc.refs++
  159. return bc
  160. }
  161. func (c *consumer) unrefBrokerConsumer(brokerWorker *brokerConsumer) {
  162. c.lock.Lock()
  163. defer c.lock.Unlock()
  164. brokerWorker.refs--
  165. if brokerWorker.refs == 0 {
  166. close(brokerWorker.input)
  167. if c.brokerConsumers[brokerWorker.broker] == brokerWorker {
  168. delete(c.brokerConsumers, brokerWorker.broker)
  169. }
  170. }
  171. }
  172. func (c *consumer) abandonBrokerConsumer(brokerWorker *brokerConsumer) {
  173. c.lock.Lock()
  174. defer c.lock.Unlock()
  175. delete(c.brokerConsumers, brokerWorker.broker)
  176. }
  177. // PartitionConsumer
  178. // PartitionConsumer processes Kafka messages from a given topic and partition. You MUST call Close()
  179. // or AsyncClose() on a PartitionConsumer to avoid leaks, it will not be garbage-collected automatically
  180. // when it passes out of scope.
  181. //
  182. // The simplest way of using a PartitionConsumer is to loop over its Messages channel using a for/range
  183. // loop. The PartitionConsumer will only stop itself in one case: when the offset being consumed is reported
  184. // as out of range by the brokers. In this case you should decide what you want to do (try a different offset,
  185. // notify a human, etc) and handle it appropriately. For all other error cases, it will just keep retrying.
  186. // By default, it logs these errors to sarama.Logger; if you want to be notified directly of all errors, set
  187. // your config's Consumer.Return.Errors to true and read from the Errors channel, using a select statement
  188. // or a separate goroutine. Check out the Consumer examples to see implementations of these different approaches.
  189. type PartitionConsumer interface {
  190. // AsyncClose initiates a shutdown of the PartitionConsumer. This method will return immediately,
  191. // after which you should wait until the 'messages' and 'errors' channel are drained.
  192. // It is required to call this function, or Close before a consumer object passes out of scope,
  193. // as it will otherwise leak memory. You must call this before calling Close on the underlying
  194. // client.
  195. AsyncClose()
  196. // Close stops the PartitionConsumer from fetching messages. It is required to call this function
  197. // (or AsyncClose) before a consumer object passes out of scope, as it will otherwise leak memory. You must
  198. // call this before calling Close on the underlying client.
  199. Close() error
  200. // Messages returns the read channel for the messages that are returned by the broker.
  201. Messages() <-chan *ConsumerMessage
  202. // Errors returns a read channel of errors that occured during consuming, if enabled. By default,
  203. // errors are logged and not returned over this channel. If you want to implement any custom errpr
  204. // handling, set your config's Consumer.Return.Errors setting to true, and read from this channel.
  205. Errors() <-chan *ConsumerError
  206. // HighWaterMarkOffset returns the high water mark offset of the partition, i.e. the offset that will
  207. // be used for the next message that will be produced. You can use this to determine how far behind
  208. // the processing is.
  209. HighWaterMarkOffset() int64
  210. }
  211. type partitionConsumer struct {
  212. consumer *consumer
  213. conf *Config
  214. topic string
  215. partition int32
  216. broker *brokerConsumer
  217. messages chan *ConsumerMessage
  218. errors chan *ConsumerError
  219. feeder chan *FetchResponse
  220. trigger chan none
  221. dying chan error
  222. fetchSize int32
  223. offset int64
  224. highWaterMarkOffset int64
  225. }
  226. func (child *partitionConsumer) sendError(err error) {
  227. cErr := &ConsumerError{
  228. Topic: child.topic,
  229. Partition: child.partition,
  230. Err: err,
  231. }
  232. if child.conf.Consumer.Return.Errors {
  233. child.errors <- cErr
  234. } else {
  235. Logger.Println(cErr)
  236. }
  237. }
  238. func (child *partitionConsumer) dispatcher() {
  239. for _ = range child.trigger {
  240. select {
  241. case <-child.dying:
  242. close(child.trigger)
  243. case <-time.After(child.conf.Consumer.Retry.Backoff):
  244. if child.broker != nil {
  245. child.consumer.unrefBrokerConsumer(child.broker)
  246. child.broker = nil
  247. }
  248. Logger.Printf("consumer/%s/%d finding new broker\n", child.topic, child.partition)
  249. if err := child.dispatch(); err != nil {
  250. child.sendError(err)
  251. child.trigger <- none{}
  252. }
  253. }
  254. }
  255. if child.broker != nil {
  256. child.consumer.unrefBrokerConsumer(child.broker)
  257. }
  258. child.consumer.removeChild(child)
  259. close(child.feeder)
  260. }
  261. func (child *partitionConsumer) dispatch() error {
  262. if err := child.consumer.client.RefreshMetadata(child.topic); err != nil {
  263. return err
  264. }
  265. var leader *Broker
  266. var err error
  267. if leader, err = child.consumer.client.Leader(child.topic, child.partition); err != nil {
  268. return err
  269. }
  270. child.broker = child.consumer.refBrokerConsumer(leader)
  271. child.broker.input <- child
  272. return nil
  273. }
  274. func (child *partitionConsumer) chooseStartingOffset(offset int64) error {
  275. newestOffset, err := child.consumer.client.GetOffset(child.topic, child.partition, OffsetNewest)
  276. if err != nil {
  277. return err
  278. }
  279. oldestOffset, err := child.consumer.client.GetOffset(child.topic, child.partition, OffsetOldest)
  280. if err != nil {
  281. return err
  282. }
  283. switch {
  284. case offset == OffsetNewest:
  285. child.offset = newestOffset
  286. case offset == OffsetOldest:
  287. child.offset = oldestOffset
  288. case offset >= oldestOffset && offset <= newestOffset:
  289. child.offset = offset
  290. default:
  291. return ErrOffsetOutOfRange
  292. }
  293. return nil
  294. }
  295. func (child *partitionConsumer) Messages() <-chan *ConsumerMessage {
  296. return child.messages
  297. }
  298. func (child *partitionConsumer) Errors() <-chan *ConsumerError {
  299. return child.errors
  300. }
  301. func (child *partitionConsumer) AsyncClose() {
  302. // this triggers whatever broker owns this child to abandon it and close its trigger channel, which causes
  303. // the dispatcher to exit its loop, which removes it from the consumer then closes its 'messages' and
  304. // 'errors' channel (alternatively, if the child is already at the dispatcher for some reason, that will
  305. // also just close itself)
  306. child.dying <- nil
  307. }
  308. func (child *partitionConsumer) Close() error {
  309. child.AsyncClose()
  310. go withRecover(func() {
  311. for _ = range child.messages {
  312. // drain
  313. }
  314. })
  315. var errors ConsumerErrors
  316. for err := range child.errors {
  317. errors = append(errors, err)
  318. }
  319. if len(errors) > 0 {
  320. return errors
  321. }
  322. return nil
  323. }
  324. func (child *partitionConsumer) HighWaterMarkOffset() int64 {
  325. return atomic.LoadInt64(&child.highWaterMarkOffset)
  326. }
  327. func (child *partitionConsumer) responseFeeder() {
  328. for response := range child.feeder {
  329. switch err := child.handleResponse(response); err {
  330. case nil:
  331. break
  332. case ErrOffsetOutOfRange:
  333. // there's no point in retrying this it will just fail the same way again
  334. // so shut it down and force the user to choose what to do
  335. Logger.Printf("consumer/%s/%d shutting down because %s\n", child.topic, child.partition, err)
  336. child.sendError(err)
  337. child.AsyncClose()
  338. case ErrUnknownTopicOrPartition, ErrNotLeaderForPartition, ErrLeaderNotAvailable:
  339. // these three are not fatal errors, but do require redispatching
  340. child.dying <- err
  341. default:
  342. // dunno, tell the user and try redispatching
  343. child.sendError(err)
  344. child.dying <- err
  345. }
  346. child.broker.acks.Done()
  347. }
  348. close(child.messages)
  349. close(child.errors)
  350. }
  351. func (child *partitionConsumer) handleResponse(response *FetchResponse) error {
  352. block := response.GetBlock(child.topic, child.partition)
  353. if block == nil {
  354. return ErrIncompleteResponse
  355. }
  356. if block.Err != ErrNoError {
  357. return block.Err
  358. }
  359. if len(block.MsgSet.Messages) == 0 {
  360. // We got no messages. If we got a trailing one then we need to ask for more data.
  361. // Otherwise we just poll again and wait for one to be produced...
  362. if block.MsgSet.PartialTrailingMessage {
  363. if child.conf.Consumer.Fetch.Max > 0 && child.fetchSize == child.conf.Consumer.Fetch.Max {
  364. // we can't ask for more data, we've hit the configured limit
  365. child.sendError(ErrMessageTooLarge)
  366. child.offset++ // skip this one so we can keep processing future messages
  367. } else {
  368. child.fetchSize *= 2
  369. if child.conf.Consumer.Fetch.Max > 0 && child.fetchSize > child.conf.Consumer.Fetch.Max {
  370. child.fetchSize = child.conf.Consumer.Fetch.Max
  371. }
  372. }
  373. }
  374. return nil
  375. }
  376. // we got messages, reset our fetch size in case it was increased for a previous request
  377. child.fetchSize = child.conf.Consumer.Fetch.Default
  378. atomic.StoreInt64(&child.highWaterMarkOffset, block.HighWaterMarkOffset)
  379. incomplete := false
  380. atLeastOne := false
  381. prelude := true
  382. for _, msgBlock := range block.MsgSet.Messages {
  383. for _, msg := range msgBlock.Messages() {
  384. if prelude && msg.Offset < child.offset {
  385. continue
  386. }
  387. prelude = false
  388. if msg.Offset >= child.offset {
  389. atLeastOne = true
  390. child.messages <- &ConsumerMessage{
  391. Topic: child.topic,
  392. Partition: child.partition,
  393. Key: msg.Msg.Key,
  394. Value: msg.Msg.Value,
  395. Offset: msg.Offset,
  396. }
  397. child.offset = msg.Offset + 1
  398. } else {
  399. incomplete = true
  400. }
  401. }
  402. }
  403. if incomplete || !atLeastOne {
  404. return ErrIncompleteResponse
  405. }
  406. return nil
  407. }
  408. // brokerConsumer
  409. type brokerConsumer struct {
  410. consumer *consumer
  411. broker *Broker
  412. input chan *partitionConsumer
  413. newSubscriptions chan []*partitionConsumer
  414. wait chan none
  415. subscriptions map[*partitionConsumer]none
  416. acks sync.WaitGroup
  417. refs int
  418. }
  419. func (c *consumer) newBrokerConsumer(broker *Broker) *brokerConsumer {
  420. bc := &brokerConsumer{
  421. consumer: c,
  422. broker: broker,
  423. input: make(chan *partitionConsumer),
  424. newSubscriptions: make(chan []*partitionConsumer),
  425. wait: make(chan none),
  426. subscriptions: make(map[*partitionConsumer]none),
  427. refs: 0,
  428. }
  429. go withRecover(bc.subscriptionManager)
  430. go withRecover(bc.subscriptionConsumer)
  431. return bc
  432. }
  433. func (bc *brokerConsumer) subscriptionManager() {
  434. var buffer []*partitionConsumer
  435. // The subscriptionManager constantly accepts new subscriptions on `input` (even when the main subscriptionConsumer
  436. // goroutine is in the middle of a network request) and batches it up. The main worker goroutine picks
  437. // up a batch of new subscriptions between every network request by reading from `newSubscriptions`, so we give
  438. // it nil if no new subscriptions are available. We also write to `wait` only when new subscriptions is available,
  439. // so the main goroutine can block waiting for work if it has none.
  440. for {
  441. if len(buffer) > 0 {
  442. select {
  443. case event, ok := <-bc.input:
  444. if !ok {
  445. goto done
  446. }
  447. buffer = append(buffer, event)
  448. case bc.newSubscriptions <- buffer:
  449. buffer = nil
  450. case bc.wait <- none{}:
  451. }
  452. } else {
  453. select {
  454. case event, ok := <-bc.input:
  455. if !ok {
  456. goto done
  457. }
  458. buffer = append(buffer, event)
  459. case bc.newSubscriptions <- nil:
  460. }
  461. }
  462. }
  463. done:
  464. close(bc.wait)
  465. if len(buffer) > 0 {
  466. bc.newSubscriptions <- buffer
  467. }
  468. close(bc.newSubscriptions)
  469. }
  470. func (bc *brokerConsumer) subscriptionConsumer() {
  471. <-bc.wait // wait for our first piece of work
  472. // the subscriptionConsumer ensures we will get nil right away if no new subscriptions is available
  473. for newSubscriptions := range bc.newSubscriptions {
  474. bc.updateSubscriptionCache(newSubscriptions)
  475. if len(bc.subscriptions) == 0 {
  476. // We're about to be shut down or we're about to receive more subscriptions.
  477. // Either way, the signal just hasn't propagated to our goroutine yet.
  478. <-bc.wait
  479. continue
  480. }
  481. response, err := bc.fetchNewMessages()
  482. if err != nil {
  483. Logger.Printf("consumer/broker/%d disconnecting due to error processing FetchRequest: %s\n", bc.broker.ID(), err)
  484. bc.abort(err)
  485. return
  486. }
  487. bc.acks.Add(len(bc.subscriptions))
  488. for child := range bc.subscriptions {
  489. child.feeder <- response
  490. }
  491. bc.acks.Wait()
  492. }
  493. }
  494. func (bc *brokerConsumer) updateSubscriptionCache(newSubscriptions []*partitionConsumer) {
  495. // take new subscriptions, and abandon subscriptions that have been closed
  496. for _, child := range newSubscriptions {
  497. bc.subscriptions[child] = none{}
  498. Logger.Printf("consumer/broker/%d added subscription to %s/%d\n", bc.broker.ID(), child.topic, child.partition)
  499. }
  500. for child := range bc.subscriptions {
  501. select {
  502. case err := <-child.dying:
  503. if err == nil {
  504. Logger.Printf("consumer/broker/%d closed dead subscription to %s/%d\n", bc.broker.ID(), child.topic, child.partition)
  505. close(child.trigger)
  506. } else {
  507. Logger.Printf("consumer/broker/%d abandoned subscription to %s/%d because %s\n", bc.broker.ID(), child.topic, child.partition, err)
  508. child.trigger <- none{}
  509. }
  510. delete(bc.subscriptions, child)
  511. default:
  512. }
  513. }
  514. }
  515. func (bc *brokerConsumer) abort(err error) {
  516. bc.consumer.abandonBrokerConsumer(bc)
  517. _ = bc.broker.Close() // we don't care about the error this might return, we already have one
  518. for child := range bc.subscriptions {
  519. child.sendError(err)
  520. child.trigger <- none{}
  521. }
  522. for newSubscription := range bc.newSubscriptions {
  523. for _, child := range newSubscription {
  524. child.sendError(err)
  525. child.trigger <- none{}
  526. }
  527. }
  528. }
  529. func (bc *brokerConsumer) fetchNewMessages() (*FetchResponse, error) {
  530. request := &FetchRequest{
  531. MinBytes: bc.consumer.conf.Consumer.Fetch.Min,
  532. MaxWaitTime: int32(bc.consumer.conf.Consumer.MaxWaitTime / time.Millisecond),
  533. }
  534. for child := range bc.subscriptions {
  535. request.AddBlock(child.topic, child.partition, child.offset, child.fetchSize)
  536. }
  537. return bc.broker.Fetch(request)
  538. }