Skip to content

Commit 77c7ccb

Browse files
authored
[Issue 456] Support chunking for big messages. (#805)
Master Issue: [#456](#456) ### Motivation Make pulsar go client support chunking to produce/consume big messages. The earlier implementation ([#717](#717)) didn't take into account many details, so I decided to reimplement it. ### Modifications - Add `internalSingleSend` to send message without batch because batch message will not be received by chunk. - Moved `BlockIfQueueFull` check from `internalSendAsync` to `internalSend` (`canAddQueue`) to ensure the normal block in chunking. - Make producer send big messages by chunking. - Add `chunkedMsgCtxMap` to store chunked messages meta and data. - Make consumer can obtain chunks and consume the big message.
1 parent bea85d4 commit 77c7ccb

14 files changed

+1586
-228
lines changed

pulsar/consumer.go

Lines changed: 10 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -198,6 +198,16 @@ type ConsumerOptions struct {
198198
// error information of the Ack method only contains errors that may occur in the Go SDK's own processing.
199199
// Default: false
200200
AckWithResponse bool
201+
202+
// MaxPendingChunkedMessage sets the maximum pending chunked messages. (default: 100)
203+
MaxPendingChunkedMessage int
204+
205+
// ExpireTimeOfIncompleteChunk sets the expiry time of discarding incomplete chunked message. (default: 60 seconds)
206+
ExpireTimeOfIncompleteChunk time.Duration
207+
208+
// AutoAckIncompleteChunk sets whether consumer auto acknowledges incomplete chunked message when it should
209+
// be removed (e.g.the chunked message pending queue is full). (default: false)
210+
AutoAckIncompleteChunk bool
201211
}
202212

203213
// Consumer is an interface that abstracts behavior of Pulsar's consumer

pulsar/consumer_impl.go

Lines changed: 57 additions & 48 deletions
Original file line numberDiff line numberDiff line change
@@ -19,7 +19,6 @@ package pulsar
1919

2020
import (
2121
"context"
22-
"errors"
2322
"fmt"
2423
"math/rand"
2524
"strconv"
@@ -37,9 +36,9 @@ const defaultNackRedeliveryDelay = 1 * time.Minute
3736

3837
type acker interface {
3938
// AckID does not handle errors returned by the Broker side, so no need to wait for doneCh to finish.
40-
AckID(id trackingMessageID) error
41-
AckIDWithResponse(id trackingMessageID) error
42-
NackID(id trackingMessageID)
39+
AckID(id MessageID) error
40+
AckIDWithResponse(id MessageID) error
41+
NackID(id MessageID)
4342
NackMsg(msg Message)
4443
}
4544

@@ -93,6 +92,14 @@ func newConsumer(client *client, options ConsumerOptions) (Consumer, error) {
9392
}
9493
}
9594

95+
if options.MaxPendingChunkedMessage == 0 {
96+
options.MaxPendingChunkedMessage = 100
97+
}
98+
99+
if options.ExpireTimeOfIncompleteChunk == 0 {
100+
options.ExpireTimeOfIncompleteChunk = time.Minute
101+
}
102+
96103
if options.NackBackoffPolicy == nil && options.EnableDefaultNackBackoffPolicy {
97104
options.NackBackoffPolicy = new(defaultNackBackoffPolicy)
98105
}
@@ -344,28 +351,31 @@ func (c *consumer) internalTopicSubscribeToPartitions() error {
344351
nackRedeliveryDelay = c.options.NackRedeliveryDelay
345352
}
346353
opts := &partitionConsumerOpts{
347-
topic: pt,
348-
consumerName: c.consumerName,
349-
subscription: c.options.SubscriptionName,
350-
subscriptionType: c.options.Type,
351-
subscriptionInitPos: c.options.SubscriptionInitialPosition,
352-
partitionIdx: idx,
353-
receiverQueueSize: receiverQueueSize,
354-
nackRedeliveryDelay: nackRedeliveryDelay,
355-
nackBackoffPolicy: c.options.NackBackoffPolicy,
356-
metadata: metadata,
357-
subProperties: subProperties,
358-
replicateSubscriptionState: c.options.ReplicateSubscriptionState,
359-
startMessageID: trackingMessageID{},
360-
subscriptionMode: durable,
361-
readCompacted: c.options.ReadCompacted,
362-
interceptors: c.options.Interceptors,
363-
maxReconnectToBroker: c.options.MaxReconnectToBroker,
364-
backoffPolicy: c.options.BackoffPolicy,
365-
keySharedPolicy: c.options.KeySharedPolicy,
366-
schema: c.options.Schema,
367-
decryption: c.options.Decryption,
368-
ackWithResponse: c.options.AckWithResponse,
354+
topic: pt,
355+
consumerName: c.consumerName,
356+
subscription: c.options.SubscriptionName,
357+
subscriptionType: c.options.Type,
358+
subscriptionInitPos: c.options.SubscriptionInitialPosition,
359+
partitionIdx: idx,
360+
receiverQueueSize: receiverQueueSize,
361+
nackRedeliveryDelay: nackRedeliveryDelay,
362+
nackBackoffPolicy: c.options.NackBackoffPolicy,
363+
metadata: metadata,
364+
subProperties: subProperties,
365+
replicateSubscriptionState: c.options.ReplicateSubscriptionState,
366+
startMessageID: trackingMessageID{},
367+
subscriptionMode: durable,
368+
readCompacted: c.options.ReadCompacted,
369+
interceptors: c.options.Interceptors,
370+
maxReconnectToBroker: c.options.MaxReconnectToBroker,
371+
backoffPolicy: c.options.BackoffPolicy,
372+
keySharedPolicy: c.options.KeySharedPolicy,
373+
schema: c.options.Schema,
374+
decryption: c.options.Decryption,
375+
ackWithResponse: c.options.AckWithResponse,
376+
maxPendingChunkedMessage: c.options.MaxPendingChunkedMessage,
377+
expireTimeOfIncompleteChunk: c.options.ExpireTimeOfIncompleteChunk,
378+
autoAckIncompleteChunk: c.options.AutoAckIncompleteChunk,
369379
}
370380
cons, err := newPartitionConsumer(c, c.client, opts, c.messageCh, c.dlq, c.metrics)
371381
ch <- ConsumerError{
@@ -456,20 +466,15 @@ func (c *consumer) Ack(msg Message) error {
456466

457467
// AckID the consumption of a single message, identified by its MessageID
458468
func (c *consumer) AckID(msgID MessageID) error {
459-
mid, ok := c.messageID(msgID)
460-
if !ok {
461-
return errors.New("failed to convert trackingMessageID")
462-
}
463-
464-
if mid.consumer != nil {
465-
return mid.Ack()
469+
if err := c.checkMsgIDPartition(msgID); err != nil {
470+
return err
466471
}
467472

468473
if c.options.AckWithResponse {
469-
return c.consumers[mid.partitionIdx].AckIDWithResponse(mid)
474+
return c.consumers[msgID.PartitionIdx()].AckIDWithResponse(msgID)
470475
}
471476

472-
return c.consumers[mid.partitionIdx].AckID(mid)
477+
return c.consumers[msgID.PartitionIdx()].AckID(msgID)
473478
}
474479

475480
// ReconsumeLater mark a message for redelivery after custom delay
@@ -529,7 +534,7 @@ func (c *consumer) Nack(msg Message) {
529534
}
530535

531536
if mid.consumer != nil {
532-
mid.Nack()
537+
mid.consumer.NackID(msg.ID())
533538
return
534539
}
535540
c.consumers[mid.partitionIdx].NackMsg(msg)
@@ -540,17 +545,11 @@ func (c *consumer) Nack(msg Message) {
540545
}
541546

542547
func (c *consumer) NackID(msgID MessageID) {
543-
mid, ok := c.messageID(msgID)
544-
if !ok {
545-
return
546-
}
547-
548-
if mid.consumer != nil {
549-
mid.Nack()
548+
if err := c.checkMsgIDPartition(msgID); err != nil {
550549
return
551550
}
552551

553-
c.consumers[mid.partitionIdx].NackID(mid)
552+
c.consumers[msgID.PartitionIdx()].NackID(msgID)
554553
}
555554

556555
func (c *consumer) Close() {
@@ -586,12 +585,11 @@ func (c *consumer) Seek(msgID MessageID) error {
586585
return newError(SeekFailed, "for partition topic, seek command should perform on the individual partitions")
587586
}
588587

589-
mid, ok := c.messageID(msgID)
590-
if !ok {
591-
return nil
588+
if err := c.checkMsgIDPartition(msgID); err != nil {
589+
return err
592590
}
593591

594-
return c.consumers[mid.partitionIdx].Seek(mid)
592+
return c.consumers[msgID.PartitionIdx()].Seek(msgID)
595593
}
596594

597595
func (c *consumer) SeekByTime(time time.Time) error {
@@ -608,6 +606,17 @@ func (c *consumer) SeekByTime(time time.Time) error {
608606
return errs
609607
}
610608

609+
func (c *consumer) checkMsgIDPartition(msgID MessageID) error {
610+
partition := msgID.PartitionIdx()
611+
if partition < 0 || int(partition) >= len(c.consumers) {
612+
c.log.Errorf("invalid partition index %d expected a partition between [0-%d]",
613+
partition, len(c.consumers))
614+
return fmt.Errorf("invalid partition index %d expected a partition between [0-%d]",
615+
partition, len(c.consumers))
616+
}
617+
return nil
618+
}
619+
611620
var r = &random{
612621
R: rand.New(rand.NewSource(time.Now().UnixNano())),
613622
}

pulsar/consumer_multitopic.go

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -137,10 +137,10 @@ func (c *multiTopicConsumer) AckID(msgID MessageID) error {
137137
}
138138

139139
if c.options.AckWithResponse {
140-
return mid.AckWithResponse()
140+
return mid.consumer.AckIDWithResponse(msgID)
141141
}
142142

143-
return mid.Ack()
143+
return mid.consumer.AckID(msgID)
144144
}
145145

146146
func (c *multiTopicConsumer) ReconsumeLater(msg Message, delay time.Duration) {
@@ -200,7 +200,7 @@ func (c *multiTopicConsumer) NackID(msgID MessageID) {
200200
return
201201
}
202202

203-
mid.Nack()
203+
mid.consumer.NackID(msgID)
204204
}
205205

206206
func (c *multiTopicConsumer) Close() {

0 commit comments

Comments
 (0)