17
17
package org .springframework .kafka .listener ;
18
18
19
19
import java .time .temporal .ValueRange ;
20
+ import java .util .HashMap ;
21
+ import java .util .Map ;
20
22
import java .util .function .BiConsumer ;
21
23
22
24
import org .apache .commons .logging .Log ;
23
25
import org .apache .kafka .clients .consumer .ConsumerRecord ;
26
+ import org .apache .kafka .common .TopicPartition ;
24
27
25
28
import org .springframework .lang .Nullable ;
26
29
33
36
*/
34
37
class FailedRecordTracker {
35
38
36
- private final ThreadLocal <FailedRecord > failures = new ThreadLocal <>(); // intentionally not static
39
+ private final ThreadLocal <Map < TopicPartition , FailedRecord > > failures = new ThreadLocal <>(); // intentionally not static
37
40
38
41
private final BiConsumer <ConsumerRecord <?, ?>, Exception > recoverer ;
39
42
@@ -60,14 +63,25 @@ boolean skip(ConsumerRecord<?, ?> record, Exception exception) {
60
63
recover (record , exception );
61
64
return true ;
62
65
}
63
- FailedRecord failedRecord = this .failures .get ();
64
- if (this .maxFailures > 0 && (failedRecord == null || newFailure (record , failedRecord ))) {
65
- this .failures .set (new FailedRecord (record .topic (), record .partition (), record .offset ()));
66
+ Map <TopicPartition , FailedRecord > map = this .failures .get ();
67
+ if (map == null ) {
68
+ this .failures .set (new HashMap <>());
69
+ map = this .failures .get ();
70
+ }
71
+ TopicPartition topicPartition = new TopicPartition (record .topic (), record .partition ());
72
+ FailedRecord failedRecord = map .get (topicPartition );
73
+ if (failedRecord == null || failedRecord .getOffset () != record .offset ()) {
74
+ failedRecord = new FailedRecord (record .offset ());
75
+ map .put (topicPartition , failedRecord );
66
76
return false ;
67
77
}
68
78
else if (this .maxFailures > 0 && failedRecord .incrementAndGet () >= this .maxFailures ) {
69
- recover (record , exception );
70
- return true ;
79
+ recover (record , exception );
80
+ map .remove (topicPartition );
81
+ if (map .isEmpty ()) {
82
+ this .failures .remove ();
83
+ }
84
+ return true ;
71
85
}
72
86
else {
73
87
return false ;
@@ -83,42 +97,22 @@ private void recover(ConsumerRecord<?, ?> record, Exception exception) {
83
97
}
84
98
}
85
99
86
- private boolean newFailure (ConsumerRecord <?, ?> record , FailedRecord failedRecord ) {
87
- return !failedRecord .getTopic ().equals (record .topic ())
88
- || failedRecord .getPartition () != record .partition ()
89
- || failedRecord .getOffset () != record .offset ();
90
- }
91
-
92
100
void clearThreadState () {
93
101
this .failures .remove ();
94
102
}
95
103
96
104
private static final class FailedRecord {
97
105
98
- private final String topic ;
99
-
100
- private final int partition ;
101
-
102
106
private final long offset ;
103
107
104
108
private int count ;
105
109
106
- FailedRecord (String topic , int partition , long offset ) {
107
- this .topic = topic ;
108
- this .partition = partition ;
110
+ FailedRecord (long offset ) {
109
111
this .offset = offset ;
110
112
this .count = 1 ;
111
113
}
112
114
113
- private String getTopic () {
114
- return this .topic ;
115
- }
116
-
117
- private int getPartition () {
118
- return this .partition ;
119
- }
120
-
121
- private long getOffset () {
115
+ long getOffset () {
122
116
return this .offset ;
123
117
}
124
118
0 commit comments