@@ -4,7 +4,6 @@ package sarama
4
4
5
5
import (
6
6
"errors"
7
- "github.com/stretchr/testify/assert"
8
7
"log"
9
8
"math"
10
9
"os"
@@ -18,6 +17,7 @@ import (
18
17
19
18
"github.com/fortytw2/leaktest"
20
19
"github.com/rcrowley/go-metrics"
20
+ "github.com/stretchr/testify/assert"
21
21
"github.com/stretchr/testify/require"
22
22
)
23
23
@@ -638,6 +638,69 @@ func TestAsyncProducerMultipleRetriesWithBackoffFunc(t *testing.T) {
638
638
}
639
639
}
640
640
641
+ func TestAsyncProducerWithExponentialBackoffDurations (t * testing.T ) {
642
+ var backoffDurations []time.Duration
643
+ var mu sync.Mutex
644
+
645
+ topic := "my_topic"
646
+ initBackoff := 100 * time .Millisecond
647
+ maxBackoff := 2 * time .Second
648
+ config := NewTestConfig ()
649
+
650
+ innerBackoffFunc := NewExponentialBackoff (initBackoff , maxBackoff )
651
+ backoffFunc := func (retries , maxRetries int ) time.Duration {
652
+ duration := innerBackoffFunc (retries , maxRetries )
653
+ mu .Lock ()
654
+ backoffDurations = append (backoffDurations , duration )
655
+ mu .Unlock ()
656
+ return duration
657
+ }
658
+
659
+ config .Producer .Flush .Messages = 5
660
+ config .Producer .Return .Successes = true
661
+ config .Producer .Retry .Max = 3
662
+ config .Producer .Retry .BackoffFunc = backoffFunc
663
+
664
+ broker := NewMockBroker (t , 1 )
665
+
666
+ metadataResponse := new (MetadataResponse )
667
+ metadataResponse .AddBroker (broker .Addr (), broker .BrokerID ())
668
+ metadataResponse .AddTopicPartition (topic , 0 , broker .BrokerID (), nil , nil , nil , ErrNoError )
669
+ broker .Returns (metadataResponse )
670
+
671
+ producer , err := NewAsyncProducer ([]string {broker .Addr ()}, config )
672
+ if err != nil {
673
+ t .Fatal (err )
674
+ }
675
+
676
+ failResponse := new (ProduceResponse )
677
+ failResponse .AddTopicPartition (topic , 0 , ErrNotLeaderForPartition )
678
+ successResponse := new (ProduceResponse )
679
+ successResponse .AddTopicPartition (topic , 0 , ErrNoError )
680
+
681
+ broker .Returns (failResponse )
682
+ broker .Returns (metadataResponse )
683
+ broker .Returns (failResponse )
684
+ broker .Returns (metadataResponse )
685
+ broker .Returns (successResponse )
686
+
687
+ for i := 0 ; i < 5 ; i ++ {
688
+ producer .Input () <- & ProducerMessage {Topic : topic , Value : StringEncoder ("test" )}
689
+ }
690
+
691
+ expectResults (t , producer , 5 , 0 )
692
+ closeProducer (t , producer )
693
+ broker .Close ()
694
+
695
+ assert .Greater (t , backoffDurations [0 ], time .Duration (0 ),
696
+ "Expected first backoff duration to be greater than 0" )
697
+ for i := 1 ; i < len (backoffDurations ); i ++ {
698
+ assert .Greater (t , backoffDurations [i ], time .Duration (0 ))
699
+ assert .GreaterOrEqual (t , backoffDurations [i ], backoffDurations [i - 1 ])
700
+ assert .LessOrEqual (t , backoffDurations [i ], maxBackoff )
701
+ }
702
+ }
703
+
641
704
// https://github.com/IBM/sarama/issues/2129
642
705
func TestAsyncProducerMultipleRetriesWithConcurrentRequests (t * testing.T ) {
643
706
// Logger = log.New(os.Stdout, "[sarama] ", log.LstdFlags)
0 commit comments