async_producer.go 30 KB

12345678910111213141516171819202122232425262728293031323334353637383940414243444546474849505152535455565758596061626364656667686970717273747576777879808182838485868788899091929394959697989910010110210310410510610710810911011111211311411511611711811912012112212312412512612712812913013113213313413513613713813914014114214314414514614714814915015115215315415515615715815916016116216316416516616716816917017117217317417517617717817918018118218318418518618718818919019119219319419519619719819920020120220320420520620720820921021121221321421521621721821922022122222322422522622722822923023123223323423523623723823924024124224324424524624724824925025125225325425525625725825926026126226326426526626726826927027127227327427527627727827928028128228328428528628728828929029129229329429529629729829930030130230330430530630730830931031131231331431531631731831932032132232332432532632732832933033133233333433533633733833934034134234334434534634734834935035135235335435535635735835936036136236336436536636736836937037137237337437537637737837938038138238338438538638738838939039139239339439539639739839940040140240340440540640740840941041141241341441541641741841942042142242342442542642742842943043143243343443543643743843944044144244344444544644744844945045145245345445545645745845946046146246346446546646746846947047147247347447547647747847948048148248348448548648748848949049149249349449549649749849950050150250350450550650750850951051151251351451551651751851952052152252352452552652752852953053153253353453553653753853954054154254354454554654754854955055155255355455555655755855956056156256356456556656756856957057157257357457557657757857958058158258358458558658758858959059159259359459559659759859960060160260360460560660760860961061161261361461561661761861962062162262362462562662762862963063163263363463563663763863964064164264364464564664764864965065165265365465565665765865966066166266366466566666766866967067167267367467567667767867968068168268368468568668768868969069169269369469569669769869970070170270370470570670770870971071171271371471571671771871972072172272372472572672772872973073173273373473573673773873974074174274374474574674774874975075175275375475575675775875976076176276376476576676776876977077177277377477577677777877978078178278378478578678778878979079179279379479579679779879980080180280380480580680780880981081181281381481581681781881982082182282382482582682782882983083183283383483583683783883984084184284384484584684784884985085185285385485585685785885986086186286386486586686786886987087187287387487587687787887988088188288388488588688788888989089189289389489589689789889990090190290390490590690790890991091191291391491591691791891992092192292392492592692792892993093193293393493593693793893994094194294394494594694794894995095195295395495595695795895996096196296396496596696796896997097197297397497597697797897998098198298398498598698798898999099199299399499599699799899910001001100210031004100510061007100810091010101110121013101410151016101710181019102010211022102310241025102610271028102910301031103210331034103510361037103810391040104110421043104410451046104710481049105010511052
  1. package sarama
  2. import (
  3. "encoding/binary"
  4. "fmt"
  5. "sync"
  6. "time"
  7. "github.com/eapache/go-resiliency/breaker"
  8. "github.com/eapache/queue"
  9. )
  10. // AsyncProducer publishes Kafka messages using a non-blocking API. It routes messages
  11. // to the correct broker for the provided topic-partition, refreshing metadata as appropriate,
  12. // and parses responses for errors. You must read from the Errors() channel or the
  13. // producer will deadlock. You must call Close() or AsyncClose() on a producer to avoid
  14. // leaks: it will not be garbage-collected automatically when it passes out of
  15. // scope.
  16. type AsyncProducer interface {
  17. // AsyncClose triggers a shutdown of the producer. The shutdown has completed
  18. // when both the Errors and Successes channels have been closed. When calling
  19. // AsyncClose, you *must* continue to read from those channels in order to
  20. // drain the results of any messages in flight.
  21. AsyncClose()
  22. // Close shuts down the producer and waits for any buffered messages to be
  23. // flushed. You must call this function before a producer object passes out of
  24. // scope, as it may otherwise leak memory. You must call this before calling
  25. // Close on the underlying client.
  26. Close() error
  27. // Input is the input channel for the user to write messages to that they
  28. // wish to send.
  29. Input() chan<- *ProducerMessage
  30. // Successes is the success output channel back to the user when Return.Successes is
  31. // enabled. If Return.Successes is true, you MUST read from this channel or the
  32. // Producer will deadlock. It is suggested that you send and read messages
  33. // together in a single select statement.
  34. Successes() <-chan *ProducerMessage
  35. // Errors is the error output channel back to the user. You MUST read from this
  36. // channel or the Producer will deadlock when the channel is full. Alternatively,
  37. // you can set Producer.Return.Errors in your config to false, which prevents
  38. // errors to be returned.
  39. Errors() <-chan *ProducerError
  40. }
  41. // transactionManager keeps the state necessary to ensure idempotent production
  42. type transactionManager struct {
  43. producerID int64
  44. producerEpoch int16
  45. sequenceNumbers map[string]int32
  46. mutex sync.Mutex
  47. }
  48. const (
  49. noProducerID = -1
  50. noProducerEpoch = -1
  51. )
  52. func (t *transactionManager) getAndIncrementSequenceNumber(topic string, partition int32) int32 {
  53. key := fmt.Sprintf("%s-%d", topic, partition)
  54. t.mutex.Lock()
  55. defer t.mutex.Unlock()
  56. sequence := t.sequenceNumbers[key]
  57. t.sequenceNumbers[key] = sequence + 1
  58. return sequence
  59. }
  60. func newTransactionManager(conf *Config, client Client) (*transactionManager, error) {
  61. txnmgr := &transactionManager{
  62. producerID: noProducerID,
  63. producerEpoch: noProducerEpoch,
  64. }
  65. if conf.Producer.Idempotent {
  66. initProducerIDResponse, err := client.InitProducerID()
  67. if err != nil {
  68. return nil, err
  69. }
  70. txnmgr.producerID = initProducerIDResponse.ProducerID
  71. txnmgr.producerEpoch = initProducerIDResponse.ProducerEpoch
  72. txnmgr.sequenceNumbers = make(map[string]int32)
  73. txnmgr.mutex = sync.Mutex{}
  74. Logger.Printf("Obtained a ProducerId: %d and ProducerEpoch: %d\n", txnmgr.producerID, txnmgr.producerEpoch)
  75. }
  76. return txnmgr, nil
  77. }
  78. type asyncProducer struct {
  79. client Client
  80. conf *Config
  81. ownClient bool
  82. errors chan *ProducerError
  83. input, successes, retries chan *ProducerMessage
  84. inFlight sync.WaitGroup
  85. brokers map[*Broker]*brokerProducer
  86. brokerRefs map[*brokerProducer]int
  87. brokerLock sync.Mutex
  88. txnmgr *transactionManager
  89. }
  90. // NewAsyncProducer creates a new AsyncProducer using the given broker addresses and configuration.
  91. func NewAsyncProducer(addrs []string, conf *Config) (AsyncProducer, error) {
  92. client, err := NewClient(addrs, conf)
  93. if err != nil {
  94. return nil, err
  95. }
  96. p, err := NewAsyncProducerFromClient(client)
  97. if err != nil {
  98. return nil, err
  99. }
  100. p.(*asyncProducer).ownClient = true
  101. return p, nil
  102. }
  103. // NewAsyncProducerFromClient creates a new Producer using the given client. It is still
  104. // necessary to call Close() on the underlying client when shutting down this producer.
  105. func NewAsyncProducerFromClient(client Client) (AsyncProducer, error) {
  106. // Check that we are not dealing with a closed Client before processing any other arguments
  107. if client.Closed() {
  108. return nil, ErrClosedClient
  109. }
  110. txnmgr, err := newTransactionManager(client.Config(), client)
  111. if err != nil {
  112. return nil, err
  113. }
  114. p := &asyncProducer{
  115. client: client,
  116. conf: client.Config(),
  117. errors: make(chan *ProducerError),
  118. input: make(chan *ProducerMessage),
  119. successes: make(chan *ProducerMessage),
  120. retries: make(chan *ProducerMessage),
  121. brokers: make(map[*Broker]*brokerProducer),
  122. brokerRefs: make(map[*brokerProducer]int),
  123. txnmgr: txnmgr,
  124. }
  125. // launch our singleton dispatchers
  126. go withRecover(p.dispatcher)
  127. go withRecover(p.retryHandler)
  128. return p, nil
  129. }
  130. type flagSet int8
  131. const (
  132. syn flagSet = 1 << iota // first message from partitionProducer to brokerProducer
  133. fin // final message from partitionProducer to brokerProducer and back
  134. shutdown // start the shutdown process
  135. )
  136. // ProducerMessage is the collection of elements passed to the Producer in order to send a message.
  137. type ProducerMessage struct {
  138. Topic string // The Kafka topic for this message.
  139. // The partitioning key for this message. Pre-existing Encoders include
  140. // StringEncoder and ByteEncoder.
  141. Key Encoder
  142. // The actual message to store in Kafka. Pre-existing Encoders include
  143. // StringEncoder and ByteEncoder.
  144. Value Encoder
  145. // The headers are key-value pairs that are transparently passed
  146. // by Kafka between producers and consumers.
  147. Headers []RecordHeader
  148. // This field is used to hold arbitrary data you wish to include so it
  149. // will be available when receiving on the Successes and Errors channels.
  150. // Sarama completely ignores this field and is only to be used for
  151. // pass-through data.
  152. Metadata interface{}
  153. // Below this point are filled in by the producer as the message is processed
  154. // Offset is the offset of the message stored on the broker. This is only
  155. // guaranteed to be defined if the message was successfully delivered and
  156. // RequiredAcks is not NoResponse.
  157. Offset int64
  158. // Partition is the partition that the message was sent to. This is only
  159. // guaranteed to be defined if the message was successfully delivered.
  160. Partition int32
  161. // Timestamp is the timestamp assigned to the message by the broker. This
  162. // is only guaranteed to be defined if the message was successfully
  163. // delivered, RequiredAcks is not NoResponse, and the Kafka broker is at
  164. // least version 0.10.0.
  165. Timestamp time.Time
  166. retries int
  167. flags flagSet
  168. expectation chan *ProducerError
  169. sequenceNumber int32
  170. }
  171. const producerMessageOverhead = 26 // the metadata overhead of CRC, flags, etc.
  172. func (m *ProducerMessage) byteSize(version int) int {
  173. var size int
  174. if version >= 2 {
  175. size = maximumRecordOverhead
  176. for _, h := range m.Headers {
  177. size += len(h.Key) + len(h.Value) + 2*binary.MaxVarintLen32
  178. }
  179. } else {
  180. size = producerMessageOverhead
  181. }
  182. if m.Key != nil {
  183. size += m.Key.Length()
  184. }
  185. if m.Value != nil {
  186. size += m.Value.Length()
  187. }
  188. return size
  189. }
  190. func (m *ProducerMessage) clear() {
  191. m.flags = 0
  192. m.retries = 0
  193. }
  194. // ProducerError is the type of error generated when the producer fails to deliver a message.
  195. // It contains the original ProducerMessage as well as the actual error value.
  196. type ProducerError struct {
  197. Msg *ProducerMessage
  198. Err error
  199. }
  200. func (pe ProducerError) Error() string {
  201. return fmt.Sprintf("kafka: Failed to produce message to topic %s: %s", pe.Msg.Topic, pe.Err)
  202. }
  203. // ProducerErrors is a type that wraps a batch of "ProducerError"s and implements the Error interface.
  204. // It can be returned from the Producer's Close method to avoid the need to manually drain the Errors channel
  205. // when closing a producer.
  206. type ProducerErrors []*ProducerError
  207. func (pe ProducerErrors) Error() string {
  208. return fmt.Sprintf("kafka: Failed to deliver %d messages.", len(pe))
  209. }
  210. func (p *asyncProducer) Errors() <-chan *ProducerError {
  211. return p.errors
  212. }
  213. func (p *asyncProducer) Successes() <-chan *ProducerMessage {
  214. return p.successes
  215. }
  216. func (p *asyncProducer) Input() chan<- *ProducerMessage {
  217. return p.input
  218. }
  219. func (p *asyncProducer) Close() error {
  220. p.AsyncClose()
  221. if p.conf.Producer.Return.Successes {
  222. go withRecover(func() {
  223. for range p.successes {
  224. }
  225. })
  226. }
  227. var errors ProducerErrors
  228. if p.conf.Producer.Return.Errors {
  229. for event := range p.errors {
  230. errors = append(errors, event)
  231. }
  232. } else {
  233. <-p.errors
  234. }
  235. if len(errors) > 0 {
  236. return errors
  237. }
  238. return nil
  239. }
  240. func (p *asyncProducer) AsyncClose() {
  241. go withRecover(p.shutdown)
  242. }
  243. // singleton
  244. // dispatches messages by topic
  245. func (p *asyncProducer) dispatcher() {
  246. handlers := make(map[string]chan<- *ProducerMessage)
  247. shuttingDown := false
  248. for msg := range p.input {
  249. if msg == nil {
  250. Logger.Println("Something tried to send a nil message, it was ignored.")
  251. continue
  252. }
  253. if msg.flags&shutdown != 0 {
  254. shuttingDown = true
  255. p.inFlight.Done()
  256. continue
  257. } else if msg.retries == 0 {
  258. if shuttingDown {
  259. // we can't just call returnError here because that decrements the wait group,
  260. // which hasn't been incremented yet for this message, and shouldn't be
  261. pErr := &ProducerError{Msg: msg, Err: ErrShuttingDown}
  262. if p.conf.Producer.Return.Errors {
  263. p.errors <- pErr
  264. } else {
  265. Logger.Println(pErr)
  266. }
  267. continue
  268. }
  269. p.inFlight.Add(1)
  270. }
  271. version := 1
  272. if p.conf.Version.IsAtLeast(V0_11_0_0) {
  273. version = 2
  274. } else if msg.Headers != nil {
  275. p.returnError(msg, ConfigurationError("Producing headers requires Kafka at least v0.11"))
  276. continue
  277. }
  278. if msg.byteSize(version) > p.conf.Producer.MaxMessageBytes {
  279. p.returnError(msg, ErrMessageSizeTooLarge)
  280. continue
  281. }
  282. handler := handlers[msg.Topic]
  283. if handler == nil {
  284. handler = p.newTopicProducer(msg.Topic)
  285. handlers[msg.Topic] = handler
  286. }
  287. handler <- msg
  288. }
  289. for _, handler := range handlers {
  290. close(handler)
  291. }
  292. }
  293. // one per topic
  294. // partitions messages, then dispatches them by partition
  295. type topicProducer struct {
  296. parent *asyncProducer
  297. topic string
  298. input <-chan *ProducerMessage
  299. breaker *breaker.Breaker
  300. handlers map[int32]chan<- *ProducerMessage
  301. partitioner Partitioner
  302. }
  303. func (p *asyncProducer) newTopicProducer(topic string) chan<- *ProducerMessage {
  304. input := make(chan *ProducerMessage, p.conf.ChannelBufferSize)
  305. tp := &topicProducer{
  306. parent: p,
  307. topic: topic,
  308. input: input,
  309. breaker: breaker.New(3, 1, 10*time.Second),
  310. handlers: make(map[int32]chan<- *ProducerMessage),
  311. partitioner: p.conf.Producer.Partitioner(topic),
  312. }
  313. go withRecover(tp.dispatch)
  314. return input
  315. }
  316. func (tp *topicProducer) dispatch() {
  317. for msg := range tp.input {
  318. if msg.retries == 0 {
  319. if err := tp.partitionMessage(msg); err != nil {
  320. tp.parent.returnError(msg, err)
  321. continue
  322. }
  323. }
  324. // All messages being retried (sent or not) have already had their retry count updated
  325. if tp.parent.conf.Producer.Idempotent && msg.retries == 0 {
  326. msg.sequenceNumber = tp.parent.txnmgr.getAndIncrementSequenceNumber(msg.Topic, msg.Partition)
  327. }
  328. handler := tp.handlers[msg.Partition]
  329. if handler == nil {
  330. handler = tp.parent.newPartitionProducer(msg.Topic, msg.Partition)
  331. tp.handlers[msg.Partition] = handler
  332. }
  333. handler <- msg
  334. }
  335. for _, handler := range tp.handlers {
  336. close(handler)
  337. }
  338. }
  339. func (tp *topicProducer) partitionMessage(msg *ProducerMessage) error {
  340. var partitions []int32
  341. err := tp.breaker.Run(func() (err error) {
  342. var requiresConsistency = false
  343. if ep, ok := tp.partitioner.(DynamicConsistencyPartitioner); ok {
  344. requiresConsistency = ep.MessageRequiresConsistency(msg)
  345. } else {
  346. requiresConsistency = tp.partitioner.RequiresConsistency()
  347. }
  348. if requiresConsistency {
  349. partitions, err = tp.parent.client.Partitions(msg.Topic)
  350. } else {
  351. partitions, err = tp.parent.client.WritablePartitions(msg.Topic)
  352. }
  353. return
  354. })
  355. if err != nil {
  356. return err
  357. }
  358. numPartitions := int32(len(partitions))
  359. if numPartitions == 0 {
  360. return ErrLeaderNotAvailable
  361. }
  362. choice, err := tp.partitioner.Partition(msg, numPartitions)
  363. if err != nil {
  364. return err
  365. } else if choice < 0 || choice >= numPartitions {
  366. return ErrInvalidPartition
  367. }
  368. msg.Partition = partitions[choice]
  369. return nil
  370. }
  371. // one per partition per topic
  372. // dispatches messages to the appropriate broker
  373. // also responsible for maintaining message order during retries
  374. type partitionProducer struct {
  375. parent *asyncProducer
  376. topic string
  377. partition int32
  378. input <-chan *ProducerMessage
  379. leader *Broker
  380. breaker *breaker.Breaker
  381. brokerProducer *brokerProducer
  382. // highWatermark tracks the "current" retry level, which is the only one where we actually let messages through,
  383. // all other messages get buffered in retryState[msg.retries].buf to preserve ordering
  384. // retryState[msg.retries].expectChaser simply tracks whether we've seen a fin message for a given level (and
  385. // therefore whether our buffer is complete and safe to flush)
  386. highWatermark int
  387. retryState []partitionRetryState
  388. }
  389. type partitionRetryState struct {
  390. buf []*ProducerMessage
  391. expectChaser bool
  392. }
  393. func (p *asyncProducer) newPartitionProducer(topic string, partition int32) chan<- *ProducerMessage {
  394. input := make(chan *ProducerMessage, p.conf.ChannelBufferSize)
  395. pp := &partitionProducer{
  396. parent: p,
  397. topic: topic,
  398. partition: partition,
  399. input: input,
  400. breaker: breaker.New(3, 1, 10*time.Second),
  401. retryState: make([]partitionRetryState, p.conf.Producer.Retry.Max+1),
  402. }
  403. go withRecover(pp.dispatch)
  404. return input
  405. }
  406. func (pp *partitionProducer) dispatch() {
  407. // try to prefetch the leader; if this doesn't work, we'll do a proper call to `updateLeader`
  408. // on the first message
  409. pp.leader, _ = pp.parent.client.Leader(pp.topic, pp.partition)
  410. if pp.leader != nil {
  411. pp.brokerProducer = pp.parent.getBrokerProducer(pp.leader)
  412. pp.parent.inFlight.Add(1) // we're generating a syn message; track it so we don't shut down while it's still inflight
  413. pp.brokerProducer.input <- &ProducerMessage{Topic: pp.topic, Partition: pp.partition, flags: syn}
  414. }
  415. for msg := range pp.input {
  416. if msg.retries > pp.highWatermark {
  417. // a new, higher, retry level; handle it and then back off
  418. pp.newHighWatermark(msg.retries)
  419. time.Sleep(pp.parent.conf.Producer.Retry.Backoff)
  420. } else if pp.highWatermark > 0 {
  421. // we are retrying something (else highWatermark would be 0) but this message is not a *new* retry level
  422. if msg.retries < pp.highWatermark {
  423. // in fact this message is not even the current retry level, so buffer it for now (unless it's a just a fin)
  424. if msg.flags&fin == fin {
  425. pp.retryState[msg.retries].expectChaser = false
  426. pp.parent.inFlight.Done() // this fin is now handled and will be garbage collected
  427. } else {
  428. pp.retryState[msg.retries].buf = append(pp.retryState[msg.retries].buf, msg)
  429. }
  430. continue
  431. } else if msg.flags&fin == fin {
  432. // this message is of the current retry level (msg.retries == highWatermark) and the fin flag is set,
  433. // meaning this retry level is done and we can go down (at least) one level and flush that
  434. pp.retryState[pp.highWatermark].expectChaser = false
  435. pp.flushRetryBuffers()
  436. pp.parent.inFlight.Done() // this fin is now handled and will be garbage collected
  437. continue
  438. }
  439. }
  440. // if we made it this far then the current msg contains real data, and can be sent to the next goroutine
  441. // without breaking any of our ordering guarantees
  442. if pp.brokerProducer == nil {
  443. if err := pp.updateLeader(); err != nil {
  444. pp.parent.returnError(msg, err)
  445. time.Sleep(pp.parent.conf.Producer.Retry.Backoff)
  446. continue
  447. }
  448. Logger.Printf("producer/leader/%s/%d selected broker %d\n", pp.topic, pp.partition, pp.leader.ID())
  449. }
  450. pp.brokerProducer.input <- msg
  451. }
  452. if pp.brokerProducer != nil {
  453. pp.parent.unrefBrokerProducer(pp.leader, pp.brokerProducer)
  454. }
  455. }
  456. func (pp *partitionProducer) newHighWatermark(hwm int) {
  457. Logger.Printf("producer/leader/%s/%d state change to [retrying-%d]\n", pp.topic, pp.partition, hwm)
  458. pp.highWatermark = hwm
  459. // send off a fin so that we know when everything "in between" has made it
  460. // back to us and we can safely flush the backlog (otherwise we risk re-ordering messages)
  461. pp.retryState[pp.highWatermark].expectChaser = true
  462. pp.parent.inFlight.Add(1) // we're generating a fin message; track it so we don't shut down while it's still inflight
  463. pp.brokerProducer.input <- &ProducerMessage{Topic: pp.topic, Partition: pp.partition, flags: fin, retries: pp.highWatermark - 1}
  464. // a new HWM means that our current broker selection is out of date
  465. Logger.Printf("producer/leader/%s/%d abandoning broker %d\n", pp.topic, pp.partition, pp.leader.ID())
  466. pp.parent.unrefBrokerProducer(pp.leader, pp.brokerProducer)
  467. pp.brokerProducer = nil
  468. }
  469. func (pp *partitionProducer) flushRetryBuffers() {
  470. Logger.Printf("producer/leader/%s/%d state change to [flushing-%d]\n", pp.topic, pp.partition, pp.highWatermark)
  471. for {
  472. pp.highWatermark--
  473. if pp.brokerProducer == nil {
  474. if err := pp.updateLeader(); err != nil {
  475. pp.parent.returnErrors(pp.retryState[pp.highWatermark].buf, err)
  476. goto flushDone
  477. }
  478. Logger.Printf("producer/leader/%s/%d selected broker %d\n", pp.topic, pp.partition, pp.leader.ID())
  479. }
  480. for _, msg := range pp.retryState[pp.highWatermark].buf {
  481. pp.brokerProducer.input <- msg
  482. }
  483. flushDone:
  484. pp.retryState[pp.highWatermark].buf = nil
  485. if pp.retryState[pp.highWatermark].expectChaser {
  486. Logger.Printf("producer/leader/%s/%d state change to [retrying-%d]\n", pp.topic, pp.partition, pp.highWatermark)
  487. break
  488. } else if pp.highWatermark == 0 {
  489. Logger.Printf("producer/leader/%s/%d state change to [normal]\n", pp.topic, pp.partition)
  490. break
  491. }
  492. }
  493. }
  494. func (pp *partitionProducer) updateLeader() error {
  495. return pp.breaker.Run(func() (err error) {
  496. if err = pp.parent.client.RefreshMetadata(pp.topic); err != nil {
  497. return err
  498. }
  499. if pp.leader, err = pp.parent.client.Leader(pp.topic, pp.partition); err != nil {
  500. return err
  501. }
  502. pp.brokerProducer = pp.parent.getBrokerProducer(pp.leader)
  503. pp.parent.inFlight.Add(1) // we're generating a syn message; track it so we don't shut down while it's still inflight
  504. pp.brokerProducer.input <- &ProducerMessage{Topic: pp.topic, Partition: pp.partition, flags: syn}
  505. return nil
  506. })
  507. }
  508. // one per broker; also constructs an associated flusher
  509. func (p *asyncProducer) newBrokerProducer(broker *Broker) *brokerProducer {
  510. var (
  511. input = make(chan *ProducerMessage)
  512. bridge = make(chan *produceSet)
  513. responses = make(chan *brokerProducerResponse)
  514. )
  515. bp := &brokerProducer{
  516. parent: p,
  517. broker: broker,
  518. input: input,
  519. output: bridge,
  520. responses: responses,
  521. buffer: newProduceSet(p),
  522. currentRetries: make(map[string]map[int32]error),
  523. }
  524. go withRecover(bp.run)
  525. // minimal bridge to make the network response `select`able
  526. go withRecover(func() {
  527. for set := range bridge {
  528. request := set.buildRequest()
  529. response, err := broker.Produce(request)
  530. responses <- &brokerProducerResponse{
  531. set: set,
  532. err: err,
  533. res: response,
  534. }
  535. }
  536. close(responses)
  537. })
  538. return bp
  539. }
  540. type brokerProducerResponse struct {
  541. set *produceSet
  542. err error
  543. res *ProduceResponse
  544. }
  545. // groups messages together into appropriately-sized batches for sending to the broker
  546. // handles state related to retries etc
  547. type brokerProducer struct {
  548. parent *asyncProducer
  549. broker *Broker
  550. input chan *ProducerMessage
  551. output chan<- *produceSet
  552. responses <-chan *brokerProducerResponse
  553. buffer *produceSet
  554. timer <-chan time.Time
  555. timerFired bool
  556. closing error
  557. currentRetries map[string]map[int32]error
  558. }
  559. func (bp *brokerProducer) run() {
  560. var output chan<- *produceSet
  561. Logger.Printf("producer/broker/%d starting up\n", bp.broker.ID())
  562. for {
  563. select {
  564. case msg := <-bp.input:
  565. if msg == nil {
  566. bp.shutdown()
  567. return
  568. }
  569. if msg.flags&syn == syn {
  570. Logger.Printf("producer/broker/%d state change to [open] on %s/%d\n",
  571. bp.broker.ID(), msg.Topic, msg.Partition)
  572. if bp.currentRetries[msg.Topic] == nil {
  573. bp.currentRetries[msg.Topic] = make(map[int32]error)
  574. }
  575. bp.currentRetries[msg.Topic][msg.Partition] = nil
  576. bp.parent.inFlight.Done()
  577. continue
  578. }
  579. if reason := bp.needsRetry(msg); reason != nil {
  580. bp.parent.retryMessage(msg, reason)
  581. if bp.closing == nil && msg.flags&fin == fin {
  582. // we were retrying this partition but we can start processing again
  583. delete(bp.currentRetries[msg.Topic], msg.Partition)
  584. Logger.Printf("producer/broker/%d state change to [closed] on %s/%d\n",
  585. bp.broker.ID(), msg.Topic, msg.Partition)
  586. }
  587. continue
  588. }
  589. if bp.buffer.wouldOverflow(msg) {
  590. if err := bp.waitForSpace(msg); err != nil {
  591. bp.parent.retryMessage(msg, err)
  592. continue
  593. }
  594. }
  595. if err := bp.buffer.add(msg); err != nil {
  596. bp.parent.returnError(msg, err)
  597. continue
  598. }
  599. if bp.parent.conf.Producer.Flush.Frequency > 0 && bp.timer == nil {
  600. bp.timer = time.After(bp.parent.conf.Producer.Flush.Frequency)
  601. }
  602. case <-bp.timer:
  603. bp.timerFired = true
  604. case output <- bp.buffer:
  605. bp.rollOver()
  606. case response := <-bp.responses:
  607. bp.handleResponse(response)
  608. }
  609. if bp.timerFired || bp.buffer.readyToFlush() {
  610. output = bp.output
  611. } else {
  612. output = nil
  613. }
  614. }
  615. }
  616. func (bp *brokerProducer) shutdown() {
  617. for !bp.buffer.empty() {
  618. select {
  619. case response := <-bp.responses:
  620. bp.handleResponse(response)
  621. case bp.output <- bp.buffer:
  622. bp.rollOver()
  623. }
  624. }
  625. close(bp.output)
  626. for response := range bp.responses {
  627. bp.handleResponse(response)
  628. }
  629. Logger.Printf("producer/broker/%d shut down\n", bp.broker.ID())
  630. }
  631. func (bp *brokerProducer) needsRetry(msg *ProducerMessage) error {
  632. if bp.closing != nil {
  633. return bp.closing
  634. }
  635. return bp.currentRetries[msg.Topic][msg.Partition]
  636. }
  637. func (bp *brokerProducer) waitForSpace(msg *ProducerMessage) error {
  638. Logger.Printf("producer/broker/%d maximum request accumulated, waiting for space\n", bp.broker.ID())
  639. for {
  640. select {
  641. case response := <-bp.responses:
  642. bp.handleResponse(response)
  643. // handling a response can change our state, so re-check some things
  644. if reason := bp.needsRetry(msg); reason != nil {
  645. return reason
  646. } else if !bp.buffer.wouldOverflow(msg) {
  647. return nil
  648. }
  649. case bp.output <- bp.buffer:
  650. bp.rollOver()
  651. return nil
  652. }
  653. }
  654. }
  655. func (bp *brokerProducer) rollOver() {
  656. bp.timer = nil
  657. bp.timerFired = false
  658. bp.buffer = newProduceSet(bp.parent)
  659. }
  660. func (bp *brokerProducer) handleResponse(response *brokerProducerResponse) {
  661. if response.err != nil {
  662. bp.handleError(response.set, response.err)
  663. } else {
  664. bp.handleSuccess(response.set, response.res)
  665. }
  666. if bp.buffer.empty() {
  667. bp.rollOver() // this can happen if the response invalidated our buffer
  668. }
  669. }
  670. func (bp *brokerProducer) handleSuccess(sent *produceSet, response *ProduceResponse) {
  671. // we iterate through the blocks in the request set, not the response, so that we notice
  672. // if the response is missing a block completely
  673. var retryTopics []string
  674. sent.eachPartition(func(topic string, partition int32, pSet *partitionSet) {
  675. if response == nil {
  676. // this only happens when RequiredAcks is NoResponse, so we have to assume success
  677. bp.parent.returnSuccesses(pSet.msgs)
  678. return
  679. }
  680. block := response.GetBlock(topic, partition)
  681. if block == nil {
  682. bp.parent.returnErrors(pSet.msgs, ErrIncompleteResponse)
  683. return
  684. }
  685. switch block.Err {
  686. // Success
  687. case ErrNoError:
  688. if bp.parent.conf.Version.IsAtLeast(V0_10_0_0) && !block.Timestamp.IsZero() {
  689. for _, msg := range pSet.msgs {
  690. msg.Timestamp = block.Timestamp
  691. }
  692. }
  693. for i, msg := range pSet.msgs {
  694. msg.Offset = block.Offset + int64(i)
  695. }
  696. bp.parent.returnSuccesses(pSet.msgs)
  697. // Duplicate
  698. case ErrDuplicateSequenceNumber:
  699. bp.parent.returnSuccesses(pSet.msgs)
  700. // Retriable errors
  701. case ErrInvalidMessage, ErrUnknownTopicOrPartition, ErrLeaderNotAvailable, ErrNotLeaderForPartition,
  702. ErrRequestTimedOut, ErrNotEnoughReplicas, ErrNotEnoughReplicasAfterAppend:
  703. retryTopics = append(retryTopics, topic)
  704. // Other non-retriable errors
  705. default:
  706. bp.parent.returnErrors(pSet.msgs, block.Err)
  707. }
  708. })
  709. if len(retryTopics) > 0 {
  710. if bp.parent.conf.Producer.Idempotent {
  711. err := bp.parent.client.RefreshMetadata(retryTopics...)
  712. if err != nil {
  713. Logger.Printf("Failed refreshing metadata because of %v\n", err)
  714. }
  715. }
  716. sent.eachPartition(func(topic string, partition int32, pSet *partitionSet) {
  717. block := response.GetBlock(topic, partition)
  718. if block == nil {
  719. // handled in the previous "eachPartition" loop
  720. return
  721. }
  722. switch block.Err {
  723. case ErrInvalidMessage, ErrUnknownTopicOrPartition, ErrLeaderNotAvailable, ErrNotLeaderForPartition,
  724. ErrRequestTimedOut, ErrNotEnoughReplicas, ErrNotEnoughReplicasAfterAppend:
  725. Logger.Printf("producer/broker/%d state change to [retrying] on %s/%d because %v\n",
  726. bp.broker.ID(), topic, partition, block.Err)
  727. if bp.currentRetries[topic] == nil {
  728. bp.currentRetries[topic] = make(map[int32]error)
  729. }
  730. bp.currentRetries[topic][partition] = block.Err
  731. if bp.parent.conf.Producer.Idempotent {
  732. go bp.parent.retryBatch(topic, partition, pSet, block.Err)
  733. } else {
  734. bp.parent.retryMessages(pSet.msgs, block.Err)
  735. }
  736. // dropping the following messages has the side effect of incrementing their retry count
  737. bp.parent.retryMessages(bp.buffer.dropPartition(topic, partition), block.Err)
  738. }
  739. })
  740. }
  741. }
  742. func (p *asyncProducer) retryBatch(topic string, partition int32, pSet *partitionSet, kerr KError) {
  743. Logger.Printf("Retrying batch for %v-%d because of %s\n", topic, partition, kerr)
  744. produceSet := newProduceSet(p)
  745. produceSet.msgs[topic] = make(map[int32]*partitionSet)
  746. produceSet.msgs[topic][partition] = pSet
  747. produceSet.bufferBytes += pSet.bufferBytes
  748. produceSet.bufferCount += len(pSet.msgs)
  749. for _, msg := range pSet.msgs {
  750. if msg.retries >= p.conf.Producer.Retry.Max {
  751. p.returnError(msg, kerr)
  752. return
  753. }
  754. msg.retries++
  755. }
  756. // it's expected that a metadata refresh has been requested prior to calling retryBatch
  757. leader, err := p.client.Leader(topic, partition)
  758. if err != nil {
  759. Logger.Printf("Failed retrying batch for %v-%d because of %v while looking up for new leader\n", topic, partition, err)
  760. for _, msg := range pSet.msgs {
  761. p.returnError(msg, kerr)
  762. }
  763. return
  764. }
  765. bp := p.getBrokerProducer(leader)
  766. bp.output <- produceSet
  767. }
  768. func (bp *brokerProducer) handleError(sent *produceSet, err error) {
  769. switch err.(type) {
  770. case PacketEncodingError:
  771. sent.eachPartition(func(topic string, partition int32, pSet *partitionSet) {
  772. bp.parent.returnErrors(pSet.msgs, err)
  773. })
  774. default:
  775. Logger.Printf("producer/broker/%d state change to [closing] because %s\n", bp.broker.ID(), err)
  776. bp.parent.abandonBrokerConnection(bp.broker)
  777. _ = bp.broker.Close()
  778. bp.closing = err
  779. sent.eachPartition(func(topic string, partition int32, pSet *partitionSet) {
  780. bp.parent.retryMessages(pSet.msgs, err)
  781. })
  782. bp.buffer.eachPartition(func(topic string, partition int32, pSet *partitionSet) {
  783. bp.parent.retryMessages(pSet.msgs, err)
  784. })
  785. bp.rollOver()
  786. }
  787. }
  788. // singleton
  789. // effectively a "bridge" between the flushers and the dispatcher in order to avoid deadlock
  790. // based on https://godoc.org/github.com/eapache/channels#InfiniteChannel
  791. func (p *asyncProducer) retryHandler() {
  792. var msg *ProducerMessage
  793. buf := queue.New()
  794. for {
  795. if buf.Length() == 0 {
  796. msg = <-p.retries
  797. } else {
  798. select {
  799. case msg = <-p.retries:
  800. case p.input <- buf.Peek().(*ProducerMessage):
  801. buf.Remove()
  802. continue
  803. }
  804. }
  805. if msg == nil {
  806. return
  807. }
  808. buf.Add(msg)
  809. }
  810. }
  811. // utility functions
  812. func (p *asyncProducer) shutdown() {
  813. Logger.Println("Producer shutting down.")
  814. p.inFlight.Add(1)
  815. p.input <- &ProducerMessage{flags: shutdown}
  816. p.inFlight.Wait()
  817. if p.ownClient {
  818. err := p.client.Close()
  819. if err != nil {
  820. Logger.Println("producer/shutdown failed to close the embedded client:", err)
  821. }
  822. }
  823. close(p.input)
  824. close(p.retries)
  825. close(p.errors)
  826. close(p.successes)
  827. }
  828. func (p *asyncProducer) returnError(msg *ProducerMessage, err error) {
  829. msg.clear()
  830. pErr := &ProducerError{Msg: msg, Err: err}
  831. if p.conf.Producer.Return.Errors {
  832. p.errors <- pErr
  833. } else {
  834. Logger.Println(pErr)
  835. }
  836. p.inFlight.Done()
  837. }
  838. func (p *asyncProducer) returnErrors(batch []*ProducerMessage, err error) {
  839. for _, msg := range batch {
  840. p.returnError(msg, err)
  841. }
  842. }
  843. func (p *asyncProducer) returnSuccesses(batch []*ProducerMessage) {
  844. for _, msg := range batch {
  845. if p.conf.Producer.Return.Successes {
  846. msg.clear()
  847. p.successes <- msg
  848. }
  849. p.inFlight.Done()
  850. }
  851. }
  852. func (p *asyncProducer) retryMessage(msg *ProducerMessage, err error) {
  853. if msg.retries >= p.conf.Producer.Retry.Max {
  854. p.returnError(msg, err)
  855. } else {
  856. msg.retries++
  857. p.retries <- msg
  858. }
  859. }
  860. func (p *asyncProducer) retryMessages(batch []*ProducerMessage, err error) {
  861. for _, msg := range batch {
  862. p.retryMessage(msg, err)
  863. }
  864. }
  865. func (p *asyncProducer) getBrokerProducer(broker *Broker) *brokerProducer {
  866. p.brokerLock.Lock()
  867. defer p.brokerLock.Unlock()
  868. bp := p.brokers[broker]
  869. if bp == nil {
  870. bp = p.newBrokerProducer(broker)
  871. p.brokers[broker] = bp
  872. p.brokerRefs[bp] = 0
  873. }
  874. p.brokerRefs[bp]++
  875. return bp
  876. }
  877. func (p *asyncProducer) unrefBrokerProducer(broker *Broker, bp *brokerProducer) {
  878. p.brokerLock.Lock()
  879. defer p.brokerLock.Unlock()
  880. p.brokerRefs[bp]--
  881. if p.brokerRefs[bp] == 0 {
  882. close(bp.input)
  883. delete(p.brokerRefs, bp)
  884. if p.brokers[broker] == bp {
  885. delete(p.brokers, broker)
  886. }
  887. }
  888. }
  889. func (p *asyncProducer) abandonBrokerConnection(broker *Broker) {
  890. p.brokerLock.Lock()
  891. defer p.brokerLock.Unlock()
  892. delete(p.brokers, broker)
  893. }