16
16
*/
17
17
package kafka .server
18
18
19
- import org .apache .kafka .common .test .api .{ClusterConfigProperty , ClusterTest , ClusterTestDefaults , Type }
20
19
import kafka .utils .TestUtils
21
20
import org .apache .kafka .common .errors .UnsupportedVersionException
22
- import org .apache .kafka .common .message .OffsetFetchRequestData
23
21
import org .apache .kafka .common .protocol .{ApiKeys , Errors }
24
- import org .apache .kafka .common .requests .JoinGroupRequest
22
+ import org .apache .kafka .common .requests .{ EndTxnRequest , JoinGroupRequest }
25
23
import org .apache .kafka .common .test .ClusterInstance
24
+ import org .apache .kafka .common .test .api .{ClusterConfigProperty , ClusterTest , ClusterTestDefaults , Type }
26
25
import org .apache .kafka .common .utils .ProducerIdAndEpoch
27
26
import org .apache .kafka .coordinator .group .GroupCoordinatorConfig
28
27
import org .apache .kafka .coordinator .transaction .TransactionLogConfig
29
- import org .junit .jupiter .api .Assertions .{assertThrows , assertTrue }
30
-
31
- import scala .jdk .CollectionConverters ._
28
+ import org .junit .jupiter .api .Assertions .{assertNotEquals , assertThrows }
32
29
33
30
@ ClusterTestDefaults (
34
31
types = Array (Type .KRAFT ),
@@ -51,6 +48,16 @@ class TxnOffsetCommitRequestTest(cluster:ClusterInstance) extends GroupCoordinat
51
48
testTxnOffsetCommit(false )
52
49
}
53
50
51
+ @ ClusterTest
52
+ def testDelayedTxnOffsetCommitWithBumpedEpochIsRejectedWithNewConsumerGroupProtocol (): Unit = {
53
+ testDelayedTxnOffsetCommitWithBumpedEpochIsRejected(true )
54
+ }
55
+
56
+ @ ClusterTest
57
+ def testDelayedTxnOffsetCommitWithBumpedEpochIsRejectedWithOldConsumerGroupProtocol (): Unit = {
58
+ testDelayedTxnOffsetCommitWithBumpedEpochIsRejected(false )
59
+ }
60
+
54
61
private def testTxnOffsetCommit (useNewProtocol : Boolean ): Unit = {
55
62
val topic = " topic"
56
63
val partition = 0
@@ -65,8 +72,8 @@ class TxnOffsetCommitRequestTest(cluster:ClusterInstance) extends GroupCoordinat
65
72
// Join the consumer group. Note that we don't heartbeat here so we must use
66
73
// a session long enough for the duration of the test.
67
74
val (memberId : String , memberEpoch : Int ) = joinConsumerGroup(groupId, useNewProtocol)
68
- assertTrue(memberId != JoinGroupRequest .UNKNOWN_MEMBER_ID )
69
- assertTrue(memberEpoch != JoinGroupRequest .UNKNOWN_GENERATION_ID )
75
+ assertNotEquals( JoinGroupRequest .UNKNOWN_MEMBER_ID , memberId )
76
+ assertNotEquals( JoinGroupRequest .UNKNOWN_GENERATION_ID , memberEpoch )
70
77
71
78
createTopic(topic, 1 )
72
79
@@ -178,7 +185,7 @@ class TxnOffsetCommitRequestTest(cluster:ClusterInstance) extends GroupCoordinat
178
185
transactionalId = transactionalId
179
186
)
180
187
181
- val originalOffset = fetchOffset(topic, partition, groupId )
188
+ val originalOffset = fetchOffset(groupId, topic, partition )
182
189
183
190
commitTxnOffset(
184
191
groupId = groupId,
@@ -207,31 +214,107 @@ class TxnOffsetCommitRequestTest(cluster:ClusterInstance) extends GroupCoordinat
207
214
208
215
TestUtils .waitUntilTrue(() =>
209
216
try {
210
- fetchOffset(topic, partition, groupId ) == expectedOffset
217
+ fetchOffset(groupId, topic, partition ) == expectedOffset
211
218
} catch {
212
219
case _ : Throwable => false
213
220
}, " txn commit offset validation failed"
214
221
)
215
222
}
216
223
217
- private def fetchOffset (
218
- topic : String ,
219
- partition : Int ,
220
- groupId : String
221
- ): Long = {
222
- val groupIdRecord = fetchOffsets(
223
- group = new OffsetFetchRequestData .OffsetFetchRequestGroup ()
224
- .setGroupId(groupId)
225
- .setTopics(List (
226
- new OffsetFetchRequestData .OffsetFetchRequestTopics ()
227
- .setName(topic)
228
- .setPartitionIndexes(List [Integer ](partition).asJava)
229
- ).asJava),
230
- requireStable = true ,
231
- version = 9
232
- )
233
- val topicRecord = groupIdRecord.topics.asScala.find(_.name == topic).head
234
- val partitionRecord = topicRecord.partitions.asScala.find(_.partitionIndex == partition).head
235
- partitionRecord.committedOffset
224
+ private def testDelayedTxnOffsetCommitWithBumpedEpochIsRejected (useNewProtocol : Boolean ): Unit = {
225
+ val topic = " topic"
226
+ val partition = 0
227
+ val transactionalId = " txn"
228
+ val groupId = " group"
229
+ val offset = 100L
230
+
231
+ // Creates the __consumer_offsets and __transaction_state topics because it won't be created automatically
232
+ // in this test because it does not use FindCoordinator API.
233
+ createOffsetsTopic()
234
+ createTransactionStateTopic()
235
+
236
+ // Join the consumer group. Note that we don't heartbeat here so we must use
237
+ // a session long enough for the duration of the test.
238
+ val (memberId : String , memberEpoch : Int ) = joinConsumerGroup(groupId, useNewProtocol)
239
+ assertNotEquals(JoinGroupRequest .UNKNOWN_MEMBER_ID , memberId)
240
+ assertNotEquals(JoinGroupRequest .UNKNOWN_GENERATION_ID , memberEpoch)
241
+
242
+ createTopic(topic, 1 )
243
+
244
+ for (version <- ApiKeys .TXN_OFFSET_COMMIT .oldestVersion to ApiKeys .TXN_OFFSET_COMMIT .latestVersion(isUnstableApiEnabled)) {
245
+ val useTV2 = version > EndTxnRequest .LAST_STABLE_VERSION_BEFORE_TRANSACTION_V2
246
+
247
+ // Initialize producer. Wait until the coordinator finishes loading.
248
+ var producerIdAndEpoch : ProducerIdAndEpoch = null
249
+ TestUtils .waitUntilTrue(() =>
250
+ try {
251
+ producerIdAndEpoch = initProducerId(
252
+ transactionalId = transactionalId,
253
+ producerIdAndEpoch = ProducerIdAndEpoch .NONE ,
254
+ expectedError = Errors .NONE
255
+ )
256
+ true
257
+ } catch {
258
+ case _ : Throwable => false
259
+ }, " initProducerId request failed"
260
+ )
261
+
262
+ addOffsetsToTxn(
263
+ groupId = groupId,
264
+ producerId = producerIdAndEpoch.producerId,
265
+ producerEpoch = producerIdAndEpoch.epoch,
266
+ transactionalId = transactionalId
267
+ )
268
+
269
+ // Complete the transaction.
270
+ endTxn(
271
+ producerId = producerIdAndEpoch.producerId,
272
+ producerEpoch = producerIdAndEpoch.epoch,
273
+ transactionalId = transactionalId,
274
+ isTransactionV2Enabled = useTV2,
275
+ committed = true ,
276
+ expectedError = Errors .NONE
277
+ )
278
+
279
+ // Start a new transaction. Wait for the previous transaction to complete.
280
+ TestUtils .waitUntilTrue(() =>
281
+ try {
282
+ addOffsetsToTxn(
283
+ groupId = groupId,
284
+ producerId = producerIdAndEpoch.producerId,
285
+ producerEpoch = if (useTV2) (producerIdAndEpoch.epoch + 1 ).toShort else producerIdAndEpoch.epoch,
286
+ transactionalId = transactionalId
287
+ )
288
+ true
289
+ } catch {
290
+ case _ : Throwable => false
291
+ }, " addOffsetsToTxn request failed"
292
+ )
293
+
294
+ // Committing offset with old epoch succeeds for TV1 and fails for TV2.
295
+ commitTxnOffset(
296
+ groupId = groupId,
297
+ memberId = if (version >= 3 ) memberId else JoinGroupRequest .UNKNOWN_MEMBER_ID ,
298
+ generationId = if (version >= 3 ) 1 else JoinGroupRequest .UNKNOWN_GENERATION_ID ,
299
+ producerId = producerIdAndEpoch.producerId,
300
+ producerEpoch = producerIdAndEpoch.epoch,
301
+ transactionalId = transactionalId,
302
+ topic = topic,
303
+ partition = partition,
304
+ offset = offset,
305
+ expectedError = if (useTV2) Errors .INVALID_PRODUCER_EPOCH else Errors .NONE ,
306
+ version = version.toShort
307
+ )
308
+
309
+ // Complete the transaction.
310
+ endTxn(
311
+ producerId = producerIdAndEpoch.producerId,
312
+ producerEpoch = if (useTV2) (producerIdAndEpoch.epoch + 1 ).toShort else producerIdAndEpoch.epoch,
313
+ transactionalId = transactionalId,
314
+ isTransactionV2Enabled = useTV2,
315
+ committed = true ,
316
+ expectedError = Errors .NONE
317
+ )
318
+ }
236
319
}
237
320
}
0 commit comments