@@ -41,9 +41,12 @@ type acker interface {
41
41
}
42
42
43
43
type consumer struct {
44
- client * client
45
- options ConsumerOptions
46
- consumers []* partitionConsumer
44
+ sync.Mutex
45
+ topic string
46
+ client * client
47
+ options ConsumerOptions
48
+ consumers []* partitionConsumer
49
+ consumerName string
47
50
48
51
// channel used to deliver message to clients
49
52
messageCh chan ConsumerMessage
@@ -52,6 +55,7 @@ type consumer struct {
52
55
closeOnce sync.Once
53
56
closeCh chan struct {}
54
57
errorCh chan error
58
+ ticker * time.Ticker
55
59
56
60
log * log.Entry
57
61
}
@@ -118,9 +122,11 @@ func newConsumer(client *client, options ConsumerOptions) (Consumer, error) {
118
122
return nil , newError (ResultInvalidTopicName , "topic name is required for consumer" )
119
123
}
120
124
121
- func internalTopicSubscribe (client * client , options ConsumerOptions , topic string ,
125
+ func newInternalConsumer (client * client , options ConsumerOptions , topic string ,
122
126
messageCh chan ConsumerMessage , dlq * dlqRouter ) (* consumer , error ) {
127
+
123
128
consumer := & consumer {
129
+ topic : topic ,
124
130
client : client ,
125
131
options : options ,
126
132
messageCh : messageCh ,
@@ -130,56 +136,108 @@ func internalTopicSubscribe(client *client, options ConsumerOptions, topic strin
130
136
log : log .WithField ("topic" , topic ),
131
137
}
132
138
133
- partitions , err := client .TopicPartitions (topic )
139
+ if options .Name != "" {
140
+ consumer .consumerName = options .Name
141
+ } else {
142
+ consumer .consumerName = generateRandomName ()
143
+ }
144
+
145
+ err := consumer .internalTopicSubscribeToPartitions ()
134
146
if err != nil {
135
147
return nil , err
136
148
}
137
149
138
- numPartitions := len (partitions )
139
- consumer .consumers = make ([]* partitionConsumer , numPartitions )
150
+ // set up timer to monitor for new partitions being added
151
+ duration := options .AutoDiscoveryPeriod
152
+ if duration <= 0 {
153
+ duration = defaultAutoDiscoveryDuration
154
+ }
155
+ consumer .ticker = time .NewTicker (duration )
156
+
157
+ go func () {
158
+ for range consumer .ticker .C {
159
+ consumer .log .Debug ("Auto discovering new partitions" )
160
+ consumer .internalTopicSubscribeToPartitions ()
161
+ }
162
+ }()
163
+
164
+ return consumer , nil
165
+ }
166
+
167
+ func (c * consumer ) internalTopicSubscribeToPartitions () error {
168
+ partitions , err := c .client .TopicPartitions (c .topic )
169
+ if err != nil {
170
+ return err
171
+ }
172
+
173
+ oldNumPartitions := 0
174
+ newNumPartitions := len (partitions )
175
+
176
+ c .Lock ()
177
+ defer c .Unlock ()
178
+ oldConsumers := c .consumers
179
+
180
+ if oldConsumers != nil {
181
+ oldNumPartitions = len (oldConsumers )
182
+ if oldNumPartitions == newNumPartitions {
183
+ c .log .Debug ("Number of partitions in topic has not changed" )
184
+ return nil
185
+ }
186
+
187
+ c .log .WithField ("old_partitions" , oldNumPartitions ).
188
+ WithField ("new_partitions" , newNumPartitions ).
189
+ Info ("Changed number of partitions in topic" )
190
+ }
191
+
192
+ c .consumers = make ([]* partitionConsumer , newNumPartitions )
193
+
194
+ // Copy over the existing consumer instances
195
+ for i := 0 ; i < oldNumPartitions ; i ++ {
196
+ c .consumers [i ] = oldConsumers [i ]
197
+ }
140
198
141
199
type ConsumerError struct {
142
200
err error
143
201
partition int
144
202
consumer * partitionConsumer
145
203
}
146
204
147
- consumerName := options .Name
148
- if consumerName == "" {
149
- consumerName = generateRandomName ()
150
- }
205
+ receiverQueueSize := c .options .ReceiverQueueSize
206
+ metadata := c .options .Properties
151
207
152
- receiverQueueSize := options .ReceiverQueueSize
153
- metadata := options .Properties
208
+ partitionsToAdd := newNumPartitions - oldNumPartitions
154
209
var wg sync.WaitGroup
155
- ch := make (chan ConsumerError , numPartitions )
156
- wg .Add (numPartitions )
157
- for partitionIdx , partitionTopic := range partitions {
210
+ ch := make (chan ConsumerError , partitionsToAdd )
211
+ wg .Add (partitionsToAdd )
212
+
213
+ for partitionIdx := oldNumPartitions ; partitionIdx < newNumPartitions ; partitionIdx ++ {
214
+ partitionTopic := partitions [partitionIdx ]
215
+
158
216
go func (idx int , pt string ) {
159
217
defer wg .Done ()
160
218
161
219
var nackRedeliveryDelay time.Duration
162
- if options .NackRedeliveryDelay == 0 {
220
+ if c . options .NackRedeliveryDelay == 0 {
163
221
nackRedeliveryDelay = defaultNackRedeliveryDelay
164
222
} else {
165
- nackRedeliveryDelay = options .NackRedeliveryDelay
223
+ nackRedeliveryDelay = c . options .NackRedeliveryDelay
166
224
}
167
225
opts := & partitionConsumerOpts {
168
226
topic : pt ,
169
- consumerName : consumerName ,
170
- subscription : options .SubscriptionName ,
171
- subscriptionType : options .Type ,
172
- subscriptionInitPos : options .SubscriptionInitialPosition ,
227
+ consumerName : c . consumerName ,
228
+ subscription : c . options .SubscriptionName ,
229
+ subscriptionType : c . options .Type ,
230
+ subscriptionInitPos : c . options .SubscriptionInitialPosition ,
173
231
partitionIdx : idx ,
174
232
receiverQueueSize : receiverQueueSize ,
175
233
nackRedeliveryDelay : nackRedeliveryDelay ,
176
234
metadata : metadata ,
177
- replicateSubscriptionState : options .ReplicateSubscriptionState ,
235
+ replicateSubscriptionState : c . options .ReplicateSubscriptionState ,
178
236
startMessageID : nil ,
179
237
subscriptionMode : durable ,
180
- readCompacted : options .ReadCompacted ,
238
+ readCompacted : c . options .ReadCompacted ,
181
239
}
182
- cons , err := newPartitionConsumer (consumer , client , opts , messageCh , dlq )
240
+ cons , err := newPartitionConsumer (c , c . client , opts , c . messageCh , c . dlq )
183
241
ch <- ConsumerError {
184
242
err : err ,
185
243
partition : idx ,
@@ -197,34 +255,37 @@ func internalTopicSubscribe(client *client, options ConsumerOptions, topic strin
197
255
if ce .err != nil {
198
256
err = ce .err
199
257
} else {
200
- consumer .consumers [ce .partition ] = ce .consumer
258
+ c .consumers [ce .partition ] = ce .consumer
201
259
}
202
260
}
203
261
204
262
if err != nil {
205
263
// Since there were some failures,
206
264
// cleanup all the partitions that succeeded in creating the consumer
207
- for _ , c := range consumer .consumers {
265
+ for _ , c := range c .consumers {
208
266
if c != nil {
209
267
c .Close ()
210
268
}
211
269
}
212
- return nil , err
270
+ return err
213
271
}
214
272
215
- return consumer , nil
273
+ return nil
216
274
}
217
275
218
276
func topicSubscribe (client * client , options ConsumerOptions , topic string ,
219
277
messageCh chan ConsumerMessage , dlqRouter * dlqRouter ) (Consumer , error ) {
220
- return internalTopicSubscribe (client , options , topic , messageCh , dlqRouter )
278
+ return newInternalConsumer (client , options , topic , messageCh , dlqRouter )
221
279
}
222
280
223
281
func (c * consumer ) Subscription () string {
224
282
return c .options .SubscriptionName
225
283
}
226
284
227
285
func (c * consumer ) Unsubscribe () error {
286
+ c .Lock ()
287
+ defer c .Unlock ()
288
+
228
289
var errMsg string
229
290
for _ , consumer := range c .consumers {
230
291
if err := consumer .Unsubscribe (); err != nil {
@@ -298,6 +359,9 @@ func (c *consumer) NackID(msgID MessageID) {
298
359
299
360
func (c * consumer ) Close () {
300
361
c .closeOnce .Do (func () {
362
+ c .Lock ()
363
+ defer c .Unlock ()
364
+
301
365
var wg sync.WaitGroup
302
366
for i := range c .consumers {
303
367
wg .Add (1 )
@@ -308,12 +372,16 @@ func (c *consumer) Close() {
308
372
}
309
373
wg .Wait ()
310
374
close (c .closeCh )
375
+ c .ticker .Stop ()
311
376
c .client .handlers .Del (c )
312
377
c .dlq .close ()
313
378
})
314
379
}
315
380
316
381
func (c * consumer ) Seek (msgID MessageID ) error {
382
+ c .Lock ()
383
+ defer c .Unlock ()
384
+
317
385
if len (c .consumers ) > 1 {
318
386
return errors .New ("for partition topic, seek command should perform on the individual partitions" )
319
387
}
@@ -327,6 +395,8 @@ func (c *consumer) Seek(msgID MessageID) error {
327
395
}
328
396
329
397
func (c * consumer ) SeekByTime (time time.Time ) error {
398
+ c .Lock ()
399
+ defer c .Unlock ()
330
400
if len (c .consumers ) > 1 {
331
401
return errors .New ("for partition topic, seek command should perform on the individual partitions" )
332
402
}
0 commit comments