forked from apache/kafka
/
KafkaApisTest.scala
6142 lines (5221 loc) · 250 KB
/
KafkaApisTest.scala
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
/*
* Licensed to the Apache Software Foundation (ASF) under one or more
* contributor license agreements. See the NOTICE file distributed with
* this work for additional information regarding copyright ownership.
* The ASF licenses this file to You under the Apache License, Version 2.0
* (the "License"); you may not use this file except in compliance with
* the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package kafka.server
import java.net.InetAddress
import java.nio.charset.StandardCharsets
import java.util
import java.util.Arrays.asList
import java.util.concurrent.{CompletableFuture, TimeUnit}
import java.util.{Collections, Optional, OptionalInt, OptionalLong, Properties}
import kafka.api.LeaderAndIsr
import kafka.cluster.Broker
import kafka.controller.{ControllerContext, KafkaController}
import kafka.coordinator.transaction.{InitProducerIdResult, TransactionCoordinator}
import kafka.network.RequestChannel
import kafka.server.QuotaFactory.QuotaManagers
import kafka.server.metadata.{ConfigRepository, KRaftMetadataCache, MockConfigRepository, ZkMetadataCache}
import kafka.utils.{Log4jController, MockTime, TestUtils}
import kafka.zk.KafkaZkClient
import org.apache.kafka.clients.admin.AlterConfigOp.OpType
import org.apache.kafka.clients.admin.{AlterConfigOp, ConfigEntry}
import org.apache.kafka.common.acl.AclOperation
import org.apache.kafka.common.config.ConfigResource
import org.apache.kafka.common.errors.UnsupportedVersionException
import org.apache.kafka.common.internals.{KafkaFutureImpl, Topic}
import org.apache.kafka.common.memory.MemoryPool
import org.apache.kafka.common.config.ConfigResource.Type.{BROKER, BROKER_LOGGER}
import org.apache.kafka.common.message.AddPartitionsToTxnRequestData.{AddPartitionsToTxnTopic, AddPartitionsToTxnTopicCollection, AddPartitionsToTxnTransaction, AddPartitionsToTxnTransactionCollection}
import org.apache.kafka.common.message.AddPartitionsToTxnResponseData.AddPartitionsToTxnResult
import org.apache.kafka.common.message.AlterConfigsRequestData.{AlterConfigsResourceCollection => LAlterConfigsResourceCollection}
import org.apache.kafka.common.message.AlterConfigsRequestData.{AlterConfigsResource => LAlterConfigsResource}
import org.apache.kafka.common.message.AlterConfigsRequestData.{AlterableConfigCollection => LAlterableConfigCollection}
import org.apache.kafka.common.message.AlterConfigsRequestData.{AlterableConfig => LAlterableConfig}
import org.apache.kafka.common.message.AlterConfigsResponseData.{AlterConfigsResourceResponse => LAlterConfigsResourceResponse}
import org.apache.kafka.common.message.ApiMessageType.ListenerType
import org.apache.kafka.common.message.CreateTopicsRequestData.{CreatableTopic, CreatableTopicCollection}
import org.apache.kafka.common.message.DescribeConfigsResponseData.DescribeConfigsResult
import org.apache.kafka.common.message.IncrementalAlterConfigsRequestData.{AlterConfigsResource => IAlterConfigsResource}
import org.apache.kafka.common.message.IncrementalAlterConfigsRequestData.{AlterConfigsResourceCollection => IAlterConfigsResourceCollection}
import org.apache.kafka.common.message.IncrementalAlterConfigsRequestData.{AlterableConfig => IAlterableConfig}
import org.apache.kafka.common.message.IncrementalAlterConfigsRequestData.{AlterableConfigCollection => IAlterableConfigCollection}
import org.apache.kafka.common.message.IncrementalAlterConfigsResponseData.{AlterConfigsResourceResponse => IAlterConfigsResourceResponse}
import org.apache.kafka.common.message.LeaveGroupRequestData.MemberIdentity
import org.apache.kafka.common.message.ListOffsetsRequestData.{ListOffsetsPartition, ListOffsetsTopic}
import org.apache.kafka.common.message.MetadataResponseData.MetadataResponseTopic
import org.apache.kafka.common.message.OffsetDeleteRequestData.{OffsetDeleteRequestPartition, OffsetDeleteRequestTopic, OffsetDeleteRequestTopicCollection}
import org.apache.kafka.common.message.StopReplicaRequestData.{StopReplicaPartitionState, StopReplicaTopicState}
import org.apache.kafka.common.message.UpdateMetadataRequestData.{UpdateMetadataBroker, UpdateMetadataEndpoint, UpdateMetadataPartitionState}
import org.apache.kafka.common.message._
import org.apache.kafka.common.metrics.Metrics
import org.apache.kafka.common.network.{ClientInformation, ListenerName}
import org.apache.kafka.common.protocol.{ApiKeys, Errors, MessageUtil}
import org.apache.kafka.common.quota.{ClientQuotaAlteration, ClientQuotaEntity}
import org.apache.kafka.common.record.FileRecords.TimestampAndOffset
import org.apache.kafka.common.record._
import org.apache.kafka.common.requests.FindCoordinatorRequest.CoordinatorType
import org.apache.kafka.common.requests.MetadataResponse.TopicMetadata
import org.apache.kafka.common.requests.ProduceResponse.PartitionResponse
import org.apache.kafka.common.requests.WriteTxnMarkersRequest.TxnMarkerEntry
import org.apache.kafka.common.requests.{FetchMetadata => JFetchMetadata, _}
import org.apache.kafka.common.resource.{PatternType, Resource, ResourcePattern, ResourceType}
import org.apache.kafka.common.security.auth.{KafkaPrincipal, KafkaPrincipalSerde, SecurityProtocol}
import org.apache.kafka.common.utils.annotation.ApiKeyVersionsSource
import org.apache.kafka.common.utils.{ProducerIdAndEpoch, SecurityUtils, Utils}
import org.apache.kafka.common.{ElectionType, IsolationLevel, Node, TopicIdPartition, TopicPartition, Uuid}
import org.apache.kafka.server.authorizer.{Action, AuthorizationResult, Authorizer}
import org.junit.jupiter.api.Assertions._
import org.junit.jupiter.api.{AfterEach, Test}
import org.junit.jupiter.params.ParameterizedTest
import org.junit.jupiter.params.provider.{CsvSource, ValueSource}
import org.mockito.ArgumentMatchers.{any, anyBoolean, anyDouble, anyInt, anyLong, anyShort, anyString, argThat, isNotNull}
import org.mockito.Mockito.{mock, reset, times, verify, when}
import org.mockito.{ArgumentCaptor, ArgumentMatchers, Mockito}
import scala.collection.{Map, Seq, mutable}
import scala.jdk.CollectionConverters._
import org.apache.kafka.common.message.CreatePartitionsRequestData.CreatePartitionsTopic
import org.apache.kafka.common.message.CreateTopicsResponseData.CreatableTopicResult
import org.apache.kafka.common.message.OffsetDeleteResponseData.{OffsetDeleteResponsePartition, OffsetDeleteResponsePartitionCollection, OffsetDeleteResponseTopic, OffsetDeleteResponseTopicCollection}
import org.apache.kafka.coordinator.group.GroupCoordinator
import org.apache.kafka.server.common.MetadataVersion
import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_2_IV0, IBP_2_2_IV1}
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchParams, FetchPartitionData}
class KafkaApisTest {
private val requestChannel: RequestChannel = mock(classOf[RequestChannel])
private val requestChannelMetrics: RequestChannel.Metrics = mock(classOf[RequestChannel.Metrics])
private val replicaManager: ReplicaManager = mock(classOf[ReplicaManager])
private val groupCoordinator: GroupCoordinator = mock(classOf[GroupCoordinator])
private val adminManager: ZkAdminManager = mock(classOf[ZkAdminManager])
private val txnCoordinator: TransactionCoordinator = mock(classOf[TransactionCoordinator])
private val controller: KafkaController = mock(classOf[KafkaController])
private val forwardingManager: ForwardingManager = mock(classOf[ForwardingManager])
private val autoTopicCreationManager: AutoTopicCreationManager = mock(classOf[AutoTopicCreationManager])
private val kafkaPrincipalSerde = new KafkaPrincipalSerde {
override def serialize(principal: KafkaPrincipal): Array[Byte] = Utils.utf8(principal.toString)
override def deserialize(bytes: Array[Byte]): KafkaPrincipal = SecurityUtils.parseKafkaPrincipal(Utils.utf8(bytes))
}
private val zkClient: KafkaZkClient = mock(classOf[KafkaZkClient])
private val metrics = new Metrics()
private val brokerId = 1
// KRaft tests should override this with a KRaftMetadataCache
private var metadataCache: MetadataCache = MetadataCache.zkMetadataCache(brokerId, MetadataVersion.latest())
private val brokerEpochManager: ZkBrokerEpochManager = new ZkBrokerEpochManager(metadataCache, controller, None)
private val clientQuotaManager: ClientQuotaManager = mock(classOf[ClientQuotaManager])
private val clientRequestQuotaManager: ClientRequestQuotaManager = mock(classOf[ClientRequestQuotaManager])
private val clientControllerQuotaManager: ControllerMutationQuotaManager = mock(classOf[ControllerMutationQuotaManager])
private val replicaQuotaManager: ReplicationQuotaManager = mock(classOf[ReplicationQuotaManager])
private val quotas = QuotaManagers(clientQuotaManager, clientQuotaManager, clientRequestQuotaManager,
clientControllerQuotaManager, replicaQuotaManager, replicaQuotaManager, replicaQuotaManager, None)
private val fetchManager: FetchManager = mock(classOf[FetchManager])
private val brokerTopicStats = new BrokerTopicStats
private val clusterId = "clusterId"
private val time = new MockTime
private val clientId = ""
@AfterEach
def tearDown(): Unit = {
quotas.shutdown()
TestUtils.clearYammerMetrics()
metrics.close()
}
def createKafkaApis(interBrokerProtocolVersion: MetadataVersion = MetadataVersion.latest,
authorizer: Option[Authorizer] = None,
enableForwarding: Boolean = false,
configRepository: ConfigRepository = new MockConfigRepository(),
raftSupport: Boolean = false,
overrideProperties: Map[String, String] = Map.empty): KafkaApis = {
val properties = if (raftSupport) {
val properties = TestUtils.createBrokerConfig(brokerId, "")
properties.put(KafkaConfig.NodeIdProp, brokerId.toString)
properties.put(KafkaConfig.ProcessRolesProp, "broker")
val voterId = brokerId + 1
properties.put(KafkaConfig.QuorumVotersProp, s"$voterId@localhost:9093")
properties.put(KafkaConfig.ControllerListenerNamesProp, "SSL")
properties
} else {
TestUtils.createBrokerConfig(brokerId, "zk")
}
overrideProperties.foreach( p => properties.put(p._1, p._2))
TestUtils.setIbpAndMessageFormatVersions(properties, interBrokerProtocolVersion)
val config = new KafkaConfig(properties)
val forwardingManagerOpt = if (enableForwarding)
Some(this.forwardingManager)
else
None
val metadataSupport = if (raftSupport) {
// it will be up to the test to replace the default ZkMetadataCache implementation
// with a KRaftMetadataCache instance
metadataCache match {
case cache: KRaftMetadataCache => RaftSupport(forwardingManager, cache)
case _ => throw new IllegalStateException("Test must set an instance of KRaftMetadataCache")
}
} else {
metadataCache match {
case zkMetadataCache: ZkMetadataCache =>
ZkSupport(adminManager, controller, zkClient, forwardingManagerOpt, zkMetadataCache, brokerEpochManager)
case _ => throw new IllegalStateException("Test must set an instance of ZkMetadataCache")
}
}
val listenerType = if (raftSupport) ListenerType.BROKER else ListenerType.ZK_BROKER
val enabledApis = if (enableForwarding) {
ApiKeys.apisForListener(listenerType).asScala ++ Set(ApiKeys.ENVELOPE)
} else {
ApiKeys.apisForListener(listenerType).asScala.toSet
}
val apiVersionManager = new SimpleApiVersionManager(listenerType, enabledApis, BrokerFeatures.defaultSupportedFeatures(), true, false)
new KafkaApis(
requestChannel = requestChannel,
metadataSupport = metadataSupport,
replicaManager = replicaManager,
groupCoordinator = groupCoordinator,
txnCoordinator = txnCoordinator,
autoTopicCreationManager = autoTopicCreationManager,
brokerId = brokerId,
config = config,
configRepository = configRepository,
metadataCache = metadataCache,
metrics = metrics,
authorizer = authorizer,
quotas = quotas,
fetchManager = fetchManager,
brokerTopicStats = brokerTopicStats,
clusterId = clusterId,
time = time,
tokenManager = null,
apiVersionManager = apiVersionManager)
}
@Test
def testDescribeConfigsWithAuthorizer(): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
val operation = AclOperation.DESCRIBE_CONFIGS
val resourceType = ResourceType.TOPIC
val resourceName = "topic-1"
val requestHeader = new RequestHeader(ApiKeys.DESCRIBE_CONFIGS, ApiKeys.DESCRIBE_CONFIGS.latestVersion,
clientId, 0)
val expectedActions = Seq(
new Action(operation, new ResourcePattern(resourceType, resourceName, PatternType.LITERAL),
1, true, true)
)
// Verify that authorize is only called once
when(authorizer.authorize(any[RequestContext], ArgumentMatchers.eq(expectedActions.asJava)))
.thenReturn(Seq(AuthorizationResult.ALLOWED).asJava)
val configRepository: ConfigRepository = mock(classOf[ConfigRepository])
val topicConfigs = new Properties()
val propName = "min.insync.replicas"
val propValue = "3"
topicConfigs.put(propName, propValue)
when(configRepository.topicConfig(resourceName)).thenReturn(topicConfigs)
metadataCache = mock(classOf[ZkMetadataCache])
when(metadataCache.contains(resourceName)).thenReturn(true)
val describeConfigsRequest = new DescribeConfigsRequest.Builder(new DescribeConfigsRequestData()
.setIncludeSynonyms(true)
.setResources(List(new DescribeConfigsRequestData.DescribeConfigsResource()
.setResourceName(resourceName)
.setResourceType(ConfigResource.Type.TOPIC.id)).asJava))
.build(requestHeader.apiVersion)
val request = buildRequest(describeConfigsRequest,
requestHeader = Option(requestHeader))
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
createKafkaApis(authorizer = Some(authorizer), configRepository = configRepository)
.handleDescribeConfigsRequest(request)
verify(authorizer).authorize(any(), ArgumentMatchers.eq(expectedActions.asJava))
val response = verifyNoThrottling[DescribeConfigsResponse](request)
val results = response.data().results()
assertEquals(1, results.size())
val describeConfigsResult: DescribeConfigsResult = results.get(0)
assertEquals(ConfigResource.Type.TOPIC.id, describeConfigsResult.resourceType())
assertEquals(resourceName, describeConfigsResult.resourceName())
val configs = describeConfigsResult.configs().asScala.filter(_.name() == propName)
assertEquals(1, configs.length)
val describeConfigsResponseData = configs.head
assertEquals(propName, describeConfigsResponseData.name())
assertEquals(propValue, describeConfigsResponseData.value())
}
@Test
def testEnvelopeRequestHandlingAsController(): Unit = {
testEnvelopeRequestWithAlterConfig(
alterConfigHandler = () => ApiError.NONE,
expectedError = Errors.NONE
)
}
@Test
def testEnvelopeRequestWithAlterConfigUnhandledError(): Unit = {
testEnvelopeRequestWithAlterConfig(
alterConfigHandler = () => throw new IllegalStateException(),
expectedError = Errors.UNKNOWN_SERVER_ERROR
)
}
private def testEnvelopeRequestWithAlterConfig(
alterConfigHandler: () => ApiError,
expectedError: Errors
): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
authorizeResource(authorizer, AclOperation.CLUSTER_ACTION, ResourceType.CLUSTER, Resource.CLUSTER_NAME, AuthorizationResult.ALLOWED)
val operation = AclOperation.ALTER_CONFIGS
val resourceName = "topic-1"
val requestHeader = new RequestHeader(ApiKeys.ALTER_CONFIGS, ApiKeys.ALTER_CONFIGS.latestVersion,
clientId, 0)
when(controller.isActive).thenReturn(true)
authorizeResource(authorizer, operation, ResourceType.TOPIC, resourceName, AuthorizationResult.ALLOWED)
val configResource = new ConfigResource(ConfigResource.Type.TOPIC, resourceName)
when(adminManager.alterConfigs(any(), ArgumentMatchers.eq(false)))
.thenAnswer(_ => {
Map(configResource -> alterConfigHandler.apply())
})
val configs = Map(
configResource -> new AlterConfigsRequest.Config(
Seq(new AlterConfigsRequest.ConfigEntry("foo", "bar")).asJava))
val alterConfigsRequest = new AlterConfigsRequest.Builder(configs.asJava, false).build(requestHeader.apiVersion)
val startTimeNanos = time.nanoseconds()
val queueDurationNanos = 5 * 1000 * 1000
val request = TestUtils.buildEnvelopeRequest(
alterConfigsRequest, kafkaPrincipalSerde, requestChannelMetrics, startTimeNanos, startTimeNanos + queueDurationNanos)
val capturedResponse: ArgumentCaptor[AlterConfigsResponse] = ArgumentCaptor.forClass(classOf[AlterConfigsResponse])
val capturedRequest: ArgumentCaptor[RequestChannel.Request] = ArgumentCaptor.forClass(classOf[RequestChannel.Request])
createKafkaApis(authorizer = Some(authorizer), enableForwarding = true).handle(request, RequestLocal.withThreadConfinedCaching)
verify(requestChannel).sendResponse(
capturedRequest.capture(),
capturedResponse.capture(),
any()
)
assertEquals(Some(request), capturedRequest.getValue.envelope)
// the dequeue time of forwarded request should equals to envelop request
assertEquals(request.requestDequeueTimeNanos, capturedRequest.getValue.requestDequeueTimeNanos)
val innerResponse = capturedResponse.getValue
val responseMap = innerResponse.data.responses().asScala.map { resourceResponse =>
resourceResponse.resourceName() -> Errors.forCode(resourceResponse.errorCode)
}.toMap
assertEquals(Map(resourceName -> expectedError), responseMap)
verify(controller).isActive
verify(adminManager).alterConfigs(any(), ArgumentMatchers.eq(false))
}
@Test
def testInvalidEnvelopeRequestWithNonForwardableAPI(): Unit = {
val requestHeader = new RequestHeader(ApiKeys.LEAVE_GROUP, ApiKeys.LEAVE_GROUP.latestVersion,
clientId, 0)
val leaveGroupRequest = new LeaveGroupRequest.Builder("group",
Collections.singletonList(new MemberIdentity())).build(requestHeader.apiVersion)
when(controller.isActive).thenReturn(true)
val request = TestUtils.buildEnvelopeRequest(
leaveGroupRequest, kafkaPrincipalSerde, requestChannelMetrics, time.nanoseconds())
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
createKafkaApis(enableForwarding = true).handle(request, RequestLocal.withThreadConfinedCaching)
val response = verifyNoThrottling[EnvelopeResponse](request)
assertEquals(Errors.INVALID_REQUEST, response.error())
}
@Test
def testEnvelopeRequestWithNotFromPrivilegedListener(): Unit = {
testInvalidEnvelopeRequest(Errors.NONE, fromPrivilegedListener = false,
shouldCloseConnection = true)
}
@Test
def testEnvelopeRequestNotAuthorized(): Unit = {
testInvalidEnvelopeRequest(Errors.CLUSTER_AUTHORIZATION_FAILED,
performAuthorize = true, authorizeResult = AuthorizationResult.DENIED)
}
@Test
def testEnvelopeRequestNotControllerHandling(): Unit = {
testInvalidEnvelopeRequest(Errors.NOT_CONTROLLER, performAuthorize = true, isActiveController = false)
}
private def testInvalidEnvelopeRequest(expectedError: Errors,
fromPrivilegedListener: Boolean = true,
shouldCloseConnection: Boolean = false,
performAuthorize: Boolean = false,
authorizeResult: AuthorizationResult = AuthorizationResult.ALLOWED,
isActiveController: Boolean = true): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
if (performAuthorize) {
authorizeResource(authorizer, AclOperation.CLUSTER_ACTION, ResourceType.CLUSTER, Resource.CLUSTER_NAME, authorizeResult)
}
val resourceName = "topic-1"
val requestHeader = new RequestHeader(ApiKeys.ALTER_CONFIGS, ApiKeys.ALTER_CONFIGS.latestVersion,
clientId, 0)
when(controller.isActive).thenReturn(isActiveController)
val configResource = new ConfigResource(ConfigResource.Type.TOPIC, resourceName)
val configs = Map(
configResource -> new AlterConfigsRequest.Config(
Seq(new AlterConfigsRequest.ConfigEntry("foo", "bar")).asJava))
val alterConfigsRequest = new AlterConfigsRequest.Builder(configs.asJava, false)
.build(requestHeader.apiVersion)
val request = TestUtils.buildEnvelopeRequest(
alterConfigsRequest, kafkaPrincipalSerde, requestChannelMetrics, time.nanoseconds(), fromPrivilegedListener = fromPrivilegedListener)
val capturedResponse: ArgumentCaptor[AbstractResponse] = ArgumentCaptor.forClass(classOf[AbstractResponse])
createKafkaApis(authorizer = Some(authorizer), enableForwarding = true).handle(request, RequestLocal.withThreadConfinedCaching)
if (shouldCloseConnection) {
verify(requestChannel).closeConnection(
ArgumentMatchers.eq(request),
ArgumentMatchers.eq(java.util.Collections.emptyMap())
)
} else {
verify(requestChannel).sendResponse(
ArgumentMatchers.eq(request),
capturedResponse.capture(),
ArgumentMatchers.eq(None))
val response = capturedResponse.getValue.asInstanceOf[EnvelopeResponse]
assertEquals(expectedError, response.error)
}
if (performAuthorize) {
verify(authorizer).authorize(any(), any())
}
}
@Test
def testAlterConfigsWithAuthorizer(): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
val authorizedTopic = "authorized-topic"
val unauthorizedTopic = "unauthorized-topic"
val (authorizedResource, unauthorizedResource) =
createConfigsWithAuthorization(authorizer, authorizedTopic, unauthorizedTopic)
val configs = Map(
authorizedResource -> new AlterConfigsRequest.Config(
Seq(new AlterConfigsRequest.ConfigEntry("foo", "bar")).asJava),
unauthorizedResource -> new AlterConfigsRequest.Config(
Seq(new AlterConfigsRequest.ConfigEntry("foo-1", "bar-1")).asJava)
)
val topicHeader = new RequestHeader(ApiKeys.ALTER_CONFIGS, ApiKeys.ALTER_CONFIGS.latestVersion,
clientId, 0)
val alterConfigsRequest = new AlterConfigsRequest.Builder(configs.asJava, false)
.build(topicHeader.apiVersion)
val request = buildRequest(alterConfigsRequest)
when(controller.isActive).thenReturn(false)
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
when(adminManager.alterConfigs(any(), ArgumentMatchers.eq(false)))
.thenReturn(Map(authorizedResource -> ApiError.NONE))
createKafkaApis(authorizer = Some(authorizer)).handleAlterConfigsRequest(request)
val response = verifyNoThrottling[AlterConfigsResponse](request)
verifyAlterConfigResult(response, Map(authorizedTopic -> Errors.NONE,
unauthorizedTopic -> Errors.TOPIC_AUTHORIZATION_FAILED))
verify(authorizer, times(2)).authorize(any(), any())
verify(adminManager).alterConfigs(any(), anyBoolean())
}
@Test
def testElectLeadersForwarding(): Unit = {
val requestBuilder = new ElectLeadersRequest.Builder(ElectionType.PREFERRED, null, 30000)
testKraftForwarding(ApiKeys.ELECT_LEADERS, requestBuilder)
}
@Test
def testDescribeQuorumNotAllowedForZkClusters(): Unit = {
val requestData = DescribeQuorumRequest.singletonRequest(KafkaRaftServer.MetadataPartition)
val requestBuilder = new DescribeQuorumRequest.Builder(requestData)
val request = buildRequest(requestBuilder.build())
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
createKafkaApis(enableForwarding = true).handle(request, RequestLocal.withThreadConfinedCaching)
val response = verifyNoThrottling[DescribeQuorumResponse](request)
assertEquals(Errors.UNKNOWN_SERVER_ERROR, Errors.forCode(response.data.errorCode))
}
@Test
def testDescribeQuorumForwardedForKRaftClusters(): Unit = {
val requestData = DescribeQuorumRequest.singletonRequest(KafkaRaftServer.MetadataPartition)
val requestBuilder = new DescribeQuorumRequest.Builder(requestData)
metadataCache = MetadataCache.kRaftMetadataCache(brokerId)
testForwardableApi(
createKafkaApis(raftSupport = true),
ApiKeys.DESCRIBE_QUORUM,
requestBuilder
)
}
private def testKraftForwarding(
apiKey: ApiKeys,
requestBuilder: AbstractRequest.Builder[_ <: AbstractRequest]
): Unit = {
metadataCache = MetadataCache.kRaftMetadataCache(brokerId)
testForwardableApi(
createKafkaApis(enableForwarding = true, raftSupport = true),
apiKey,
requestBuilder
)
}
private def testForwardableApi(apiKey: ApiKeys, requestBuilder: AbstractRequest.Builder[_ <: AbstractRequest]): Unit = {
testForwardableApi(
createKafkaApis(enableForwarding = true),
apiKey,
requestBuilder
)
}
private def testForwardableApi(
kafkaApis: KafkaApis,
apiKey: ApiKeys,
requestBuilder: AbstractRequest.Builder[_ <: AbstractRequest]
): Unit = {
val topicHeader = new RequestHeader(apiKey, apiKey.latestVersion,
clientId, 0)
val apiRequest = requestBuilder.build(topicHeader.apiVersion)
val request = buildRequest(apiRequest)
if (kafkaApis.metadataSupport.isInstanceOf[ZkSupport]) {
// The controller check only makes sense for ZK clusters. For KRaft,
// controller requests are handled on a separate listener, so there
// is no choice but to forward them.
when(controller.isActive).thenReturn(false)
}
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
val forwardCallback: ArgumentCaptor[Option[AbstractResponse] => Unit] = ArgumentCaptor.forClass(classOf[Option[AbstractResponse] => Unit])
kafkaApis.handle(request, RequestLocal.withThreadConfinedCaching)
verify(forwardingManager).forwardRequest(
ArgumentMatchers.eq(request),
forwardCallback.capture()
)
assertNotNull(request.buffer, "The buffer was unexpectedly deallocated after " +
s"`handle` returned (is $apiKey marked as forwardable in `ApiKeys`?)")
val expectedResponse = apiRequest.getErrorResponse(Errors.NOT_CONTROLLER.exception)
forwardCallback.getValue.apply(Some(expectedResponse))
val capturedResponse = verifyNoThrottling[AbstractResponse](request)
assertEquals(expectedResponse.data, capturedResponse.data)
if (kafkaApis.metadataSupport.isInstanceOf[ZkSupport]) {
verify(controller).isActive
}
}
private def authorizeResource(authorizer: Authorizer,
operation: AclOperation,
resourceType: ResourceType,
resourceName: String,
result: AuthorizationResult,
logIfAllowed: Boolean = true,
logIfDenied: Boolean = true): Unit = {
val expectedAuthorizedAction = if (operation == AclOperation.CLUSTER_ACTION)
new Action(operation,
new ResourcePattern(ResourceType.CLUSTER, Resource.CLUSTER_NAME, PatternType.LITERAL),
1, logIfAllowed, logIfDenied)
else
new Action(operation,
new ResourcePattern(resourceType, resourceName, PatternType.LITERAL),
1, logIfAllowed, logIfDenied)
when(authorizer.authorize(any[RequestContext], ArgumentMatchers.eq(Seq(expectedAuthorizedAction).asJava)))
.thenReturn(Seq(result).asJava)
}
private def verifyAlterConfigResult(response: AlterConfigsResponse,
expectedResults: Map[String, Errors]): Unit = {
val responseMap = response.data.responses().asScala.map { resourceResponse =>
resourceResponse.resourceName() -> Errors.forCode(resourceResponse.errorCode)
}.toMap
assertEquals(expectedResults, responseMap)
}
private def createConfigsWithAuthorization(authorizer: Authorizer,
authorizedTopic: String,
unauthorizedTopic: String): (ConfigResource, ConfigResource) = {
val authorizedResource = new ConfigResource(ConfigResource.Type.TOPIC, authorizedTopic)
val unauthorizedResource = new ConfigResource(ConfigResource.Type.TOPIC, unauthorizedTopic)
createTopicAuthorization(authorizer, AclOperation.ALTER_CONFIGS, authorizedTopic, unauthorizedTopic)
(authorizedResource, unauthorizedResource)
}
@Test
def testIncrementalAlterConfigsWithAuthorizer(): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
val authorizedTopic = "authorized-topic"
val unauthorizedTopic = "unauthorized-topic"
val (authorizedResource, unauthorizedResource) =
createConfigsWithAuthorization(authorizer, authorizedTopic, unauthorizedTopic)
val requestHeader = new RequestHeader(ApiKeys.INCREMENTAL_ALTER_CONFIGS, ApiKeys.INCREMENTAL_ALTER_CONFIGS.latestVersion, clientId, 0)
val incrementalAlterConfigsRequest = getIncrementalAlterConfigRequestBuilder(Seq(authorizedResource, unauthorizedResource))
.build(requestHeader.apiVersion)
val request = buildRequest(incrementalAlterConfigsRequest,
fromPrivilegedListener = true, requestHeader = Option(requestHeader))
when(controller.isActive).thenReturn(true)
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
when(adminManager.incrementalAlterConfigs(any(), ArgumentMatchers.eq(false)))
.thenReturn(Map(authorizedResource -> ApiError.NONE))
createKafkaApis(authorizer = Some(authorizer)).handleIncrementalAlterConfigsRequest(request)
val capturedResponse = verifyNoThrottling[IncrementalAlterConfigsResponse](request)
verifyIncrementalAlterConfigResult(capturedResponse, Map(
authorizedTopic -> Errors.NONE,
unauthorizedTopic -> Errors.TOPIC_AUTHORIZATION_FAILED
))
verify(authorizer, times(2)).authorize(any(), any())
verify(adminManager).incrementalAlterConfigs(any(), anyBoolean())
}
private def getIncrementalAlterConfigRequestBuilder(configResources: Seq[ConfigResource]): IncrementalAlterConfigsRequest.Builder = {
val resourceMap = configResources.map(configResource => {
configResource -> Set(
new AlterConfigOp(new ConfigEntry("foo", "bar"),
OpType.forId(configResource.`type`.id))).asJavaCollection
}).toMap.asJava
new IncrementalAlterConfigsRequest.Builder(resourceMap, false)
}
private def verifyIncrementalAlterConfigResult(response: IncrementalAlterConfigsResponse,
expectedResults: Map[String, Errors]): Unit = {
val responseMap = response.data.responses().asScala.map { resourceResponse =>
resourceResponse.resourceName() -> Errors.forCode(resourceResponse.errorCode)
}.toMap
assertEquals(expectedResults, responseMap)
}
@Test
def testAlterClientQuotasWithAuthorizer(): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
authorizeResource(authorizer, AclOperation.ALTER_CONFIGS, ResourceType.CLUSTER,
Resource.CLUSTER_NAME, AuthorizationResult.DENIED)
val quotaEntity = new ClientQuotaEntity(Collections.singletonMap(ClientQuotaEntity.USER, "user"))
val quotas = Seq(new ClientQuotaAlteration(quotaEntity, Seq.empty.asJavaCollection))
val requestHeader = new RequestHeader(ApiKeys.ALTER_CLIENT_QUOTAS, ApiKeys.ALTER_CLIENT_QUOTAS.latestVersion, clientId, 0)
val alterClientQuotasRequest = new AlterClientQuotasRequest.Builder(quotas.asJavaCollection, false)
.build(requestHeader.apiVersion)
val request = buildRequest(alterClientQuotasRequest,
fromPrivilegedListener = true, requestHeader = Option(requestHeader))
when(controller.isActive).thenReturn(true)
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
anyLong)).thenReturn(0)
createKafkaApis(authorizer = Some(authorizer)).handleAlterClientQuotasRequest(request)
val capturedResponse = verifyNoThrottling[AlterClientQuotasResponse](request)
verifyAlterClientQuotaResult(capturedResponse, Map(quotaEntity -> Errors.CLUSTER_AUTHORIZATION_FAILED))
verify(authorizer).authorize(any(), any())
verify(clientRequestQuotaManager).maybeRecordAndGetThrottleTimeMs(any(), anyLong)
}
@Test
def testAlterClientQuotasWithForwarding(): Unit = {
val requestBuilder = new AlterClientQuotasRequest.Builder(List.empty.asJava, false)
testForwardableApi(ApiKeys.ALTER_CLIENT_QUOTAS, requestBuilder)
}
private def verifyAlterClientQuotaResult(response: AlterClientQuotasResponse,
expected: Map[ClientQuotaEntity, Errors]): Unit = {
val futures = expected.keys.map(quotaEntity => quotaEntity -> new KafkaFutureImpl[Void]()).toMap
response.complete(futures.asJava)
futures.foreach {
case (entity, future) =>
future.whenComplete((_, thrown) =>
assertEquals(thrown, expected(entity).exception())
).isDone
}
}
@Test
def testCreateTopicsWithAuthorizer(): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
val authorizedTopic = "authorized-topic"
val unauthorizedTopic = "unauthorized-topic"
authorizeResource(authorizer, AclOperation.CREATE, ResourceType.CLUSTER,
Resource.CLUSTER_NAME, AuthorizationResult.DENIED, logIfDenied = false)
createCombinedTopicAuthorization(authorizer, AclOperation.CREATE,
authorizedTopic, unauthorizedTopic)
createCombinedTopicAuthorization(authorizer, AclOperation.DESCRIBE_CONFIGS,
authorizedTopic, unauthorizedTopic, logIfDenied = false)
val requestHeader = new RequestHeader(ApiKeys.CREATE_TOPICS, ApiKeys.CREATE_TOPICS.latestVersion, clientId, 0)
when(controller.isActive).thenReturn(true)
val topics = new CreateTopicsRequestData.CreatableTopicCollection(2)
val topicToCreate = new CreateTopicsRequestData.CreatableTopic()
.setName(authorizedTopic)
topics.add(topicToCreate)
val topicToFilter = new CreateTopicsRequestData.CreatableTopic()
.setName(unauthorizedTopic)
topics.add(topicToFilter)
val timeout = 10
val createTopicsRequest = new CreateTopicsRequest.Builder(
new CreateTopicsRequestData()
.setTimeoutMs(timeout)
.setValidateOnly(false)
.setTopics(topics))
.build(requestHeader.apiVersion)
val request = buildRequest(createTopicsRequest,
fromPrivilegedListener = true, requestHeader = Option(requestHeader))
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(any[RequestChannel.Request](),
any[Long])).thenReturn(0)
when(clientControllerQuotaManager.newQuotaFor(
ArgumentMatchers.eq(request), ArgumentMatchers.eq(6))).thenReturn(UnboundedControllerMutationQuota)
createKafkaApis(authorizer = Some(authorizer)).handleCreateTopicsRequest(request)
val capturedCallback: ArgumentCaptor[Map[String, ApiError] => Unit] = ArgumentCaptor.forClass(classOf[Map[String, ApiError] => Unit])
verify(adminManager).createTopics(
ArgumentMatchers.eq(timeout),
ArgumentMatchers.eq(false),
ArgumentMatchers.eq(Map(authorizedTopic -> topicToCreate)),
any(),
ArgumentMatchers.eq(UnboundedControllerMutationQuota),
capturedCallback.capture())
capturedCallback.getValue.apply(Map(authorizedTopic -> ApiError.NONE))
val capturedResponse = verifyNoThrottling[CreateTopicsResponse](request)
verifyCreateTopicsResult(capturedResponse, Map(authorizedTopic -> Errors.NONE,
unauthorizedTopic -> Errors.TOPIC_AUTHORIZATION_FAILED))
}
@Test
def testCreateTopicsWithForwarding(): Unit = {
val requestBuilder = new CreateTopicsRequest.Builder(
new CreateTopicsRequestData().setTopics(
new CreatableTopicCollection(Collections.singleton(
new CreatableTopic().setName("topic").setNumPartitions(1).
setReplicationFactor(1.toShort)).iterator())))
testForwardableApi(ApiKeys.CREATE_TOPICS, requestBuilder)
}
@ParameterizedTest
@CsvSource(value = Array("0,1500", "1500,0", "3000,1000"))
def testKRaftControllerThrottleTimeEnforced(
controllerThrottleTimeMs: Int,
requestThrottleTimeMs: Int
): Unit = {
metadataCache = MetadataCache.kRaftMetadataCache(brokerId)
val topicToCreate = new CreatableTopic()
.setName("topic")
.setNumPartitions(1)
.setReplicationFactor(1.toShort)
val requestData = new CreateTopicsRequestData()
requestData.topics().add(topicToCreate)
val requestBuilder = new CreateTopicsRequest.Builder(requestData).build()
val request = buildRequest(requestBuilder)
val kafkaApis = createKafkaApis(enableForwarding = true, raftSupport = true)
val forwardCallback: ArgumentCaptor[Option[AbstractResponse] => Unit] =
ArgumentCaptor.forClass(classOf[Option[AbstractResponse] => Unit])
when(clientRequestQuotaManager.maybeRecordAndGetThrottleTimeMs(request, time.milliseconds()))
.thenReturn(requestThrottleTimeMs)
kafkaApis.handle(request, RequestLocal.withThreadConfinedCaching)
verify(forwardingManager).forwardRequest(
ArgumentMatchers.eq(request),
forwardCallback.capture()
)
val responseData = new CreateTopicsResponseData()
.setThrottleTimeMs(controllerThrottleTimeMs)
responseData.topics().add(new CreatableTopicResult()
.setErrorCode(Errors.THROTTLING_QUOTA_EXCEEDED.code))
forwardCallback.getValue.apply(Some(new CreateTopicsResponse(responseData)))
val expectedThrottleTimeMs = math.max(controllerThrottleTimeMs, requestThrottleTimeMs)
verify(clientRequestQuotaManager).throttle(
ArgumentMatchers.eq(request),
any[ThrottleCallback](),
ArgumentMatchers.eq(expectedThrottleTimeMs)
)
assertEquals(expectedThrottleTimeMs, responseData.throttleTimeMs)
}
@Test
def testCreatePartitionsAuthorization(): Unit = {
val authorizer: Authorizer = mock(classOf[Authorizer])
val kafkaApis = createKafkaApis(authorizer = Some(authorizer))
val timeoutMs = 35000
val requestData = new CreatePartitionsRequestData()
.setTimeoutMs(timeoutMs)
.setValidateOnly(false)
val fooCreatePartitionsData = new CreatePartitionsTopic().setName("foo").setAssignments(null).setCount(2)
val barCreatePartitionsData = new CreatePartitionsTopic().setName("bar").setAssignments(null).setCount(10)
requestData.topics().add(fooCreatePartitionsData)
requestData.topics().add(barCreatePartitionsData)
val fooResource = new ResourcePattern(ResourceType.TOPIC, "foo", PatternType.LITERAL)
val fooAction = new Action(AclOperation.ALTER, fooResource, 1, true, true)
val barResource = new ResourcePattern(ResourceType.TOPIC, "bar", PatternType.LITERAL)
val barAction = new Action(AclOperation.ALTER, barResource, 1, true, true)
when(authorizer.authorize(
any[RequestContext](),
any[util.List[Action]]()
)).thenAnswer { invocation =>
val actions = invocation.getArgument[util.List[Action]](1).asScala
val results = actions.map { action =>
if (action == fooAction) AuthorizationResult.ALLOWED
else if (action == barAction) AuthorizationResult.DENIED
else throw new AssertionError(s"Unexpected action $action")
}
new util.ArrayList[AuthorizationResult](results.asJava)
}
val request = buildRequest(new CreatePartitionsRequest.Builder(requestData).build())
when(controller.isActive).thenReturn(true)
when(controller.isTopicQueuedForDeletion("foo")).thenReturn(false)
when(clientControllerQuotaManager.newQuotaFor(
ArgumentMatchers.eq(request), ArgumentMatchers.anyShort())
).thenReturn(UnboundedControllerMutationQuota)
when(adminManager.createPartitions(
timeoutMs = ArgumentMatchers.eq(timeoutMs),
newPartitions = ArgumentMatchers.eq(Seq(fooCreatePartitionsData)),
validateOnly = ArgumentMatchers.eq(false),
controllerMutationQuota = ArgumentMatchers.eq(UnboundedControllerMutationQuota),
callback = ArgumentMatchers.any[Map[String, ApiError] => Unit]()
)).thenAnswer { invocation =>
val callback = invocation.getArgument[Map[String, ApiError] => Unit](4)
callback.apply(Map("foo" -> ApiError.NONE))
}
kafkaApis.handle(request, RequestLocal.withThreadConfinedCaching)
val response = verifyNoThrottling[CreatePartitionsResponse](request)
val results = response.data.results.asScala
assertEquals(Some(Errors.NONE), results.find(_.name == "foo").map(result => Errors.forCode(result.errorCode)))
assertEquals(Some(Errors.TOPIC_AUTHORIZATION_FAILED), results.find(_.name == "bar").map(result => Errors.forCode(result.errorCode)))
}
private def createTopicAuthorization(authorizer: Authorizer,
operation: AclOperation,
authorizedTopic: String,
unauthorizedTopic: String,
logIfAllowed: Boolean = true,
logIfDenied: Boolean = true): Unit = {
authorizeResource(authorizer, operation, ResourceType.TOPIC,
authorizedTopic, AuthorizationResult.ALLOWED, logIfAllowed, logIfDenied)
authorizeResource(authorizer, operation, ResourceType.TOPIC,
unauthorizedTopic, AuthorizationResult.DENIED, logIfAllowed, logIfDenied)
}
private def createCombinedTopicAuthorization(authorizer: Authorizer,
operation: AclOperation,
authorizedTopic: String,
unauthorizedTopic: String,
logIfAllowed: Boolean = true,
logIfDenied: Boolean = true): Unit = {
val expectedAuthorizedActions = Seq(
new Action(operation,
new ResourcePattern(ResourceType.TOPIC, authorizedTopic, PatternType.LITERAL),
1, logIfAllowed, logIfDenied),
new Action(operation,
new ResourcePattern(ResourceType.TOPIC, unauthorizedTopic, PatternType.LITERAL),
1, logIfAllowed, logIfDenied))
when(authorizer.authorize(
any[RequestContext], argThat((t: java.util.List[Action]) => t != null && t.containsAll(expectedAuthorizedActions.asJava))
)).thenAnswer { invocation =>
val actions = invocation.getArgument(1).asInstanceOf[util.List[Action]]
actions.asScala.map { action =>
if (action.resourcePattern().name().equals(authorizedTopic))
AuthorizationResult.ALLOWED
else
AuthorizationResult.DENIED
}.asJava
}
}
private def verifyCreateTopicsResult(response: CreateTopicsResponse,
expectedResults: Map[String, Errors]): Unit = {
val responseMap = response.data.topics().asScala.map { topicResponse =>
topicResponse.name() -> Errors.forCode(topicResponse.errorCode)
}.toMap
assertEquals(expectedResults, responseMap)
}
@Test
def testCreateAclWithForwarding(): Unit = {
val requestBuilder = new CreateAclsRequest.Builder(new CreateAclsRequestData())
testForwardableApi(ApiKeys.CREATE_ACLS, requestBuilder)
}
@Test
def testDeleteAclWithForwarding(): Unit = {
val requestBuilder = new DeleteAclsRequest.Builder(new DeleteAclsRequestData())
testForwardableApi(ApiKeys.DELETE_ACLS, requestBuilder)
}
@Test
def testCreateDelegationTokenWithForwarding(): Unit = {
val requestBuilder = new CreateDelegationTokenRequest.Builder(new CreateDelegationTokenRequestData())
testForwardableApi(ApiKeys.CREATE_DELEGATION_TOKEN, requestBuilder)
}
@Test
def testRenewDelegationTokenWithForwarding(): Unit = {
val requestBuilder = new RenewDelegationTokenRequest.Builder(new RenewDelegationTokenRequestData())
testForwardableApi(ApiKeys.RENEW_DELEGATION_TOKEN, requestBuilder)
}
@Test
def testExpireDelegationTokenWithForwarding(): Unit = {
val requestBuilder = new ExpireDelegationTokenRequest.Builder(new ExpireDelegationTokenRequestData())
testForwardableApi(ApiKeys.EXPIRE_DELEGATION_TOKEN, requestBuilder)
}
@Test
def testAlterPartitionReassignmentsWithForwarding(): Unit = {
val requestBuilder = new AlterPartitionReassignmentsRequest.Builder(new AlterPartitionReassignmentsRequestData())
testForwardableApi(ApiKeys.ALTER_PARTITION_REASSIGNMENTS, requestBuilder)
}
@Test
def testCreatePartitionsWithForwarding(): Unit = {
val requestBuilder = new CreatePartitionsRequest.Builder(new CreatePartitionsRequestData())
testForwardableApi(ApiKeys.CREATE_PARTITIONS, requestBuilder)
}
@Test
def testUpdateFeaturesWithForwarding(): Unit = {
val requestBuilder = new UpdateFeaturesRequest.Builder(new UpdateFeaturesRequestData())
testForwardableApi(ApiKeys.UPDATE_FEATURES, requestBuilder)
}
@Test
def testDeleteTopicsWithForwarding(): Unit = {
val requestBuilder = new DeleteTopicsRequest.Builder(new DeleteTopicsRequestData())
testForwardableApi(ApiKeys.DELETE_TOPICS, requestBuilder)
}
@Test
def testAlterScramWithForwarding(): Unit = {
val requestBuilder = new AlterUserScramCredentialsRequest.Builder(new AlterUserScramCredentialsRequestData())
testForwardableApi(ApiKeys.ALTER_USER_SCRAM_CREDENTIALS, requestBuilder)
}
@Test
def testFindCoordinatorAutoTopicCreationForOffsetTopic(): Unit = {
testFindCoordinatorWithTopicCreation(CoordinatorType.GROUP)
}
@Test
def testFindCoordinatorAutoTopicCreationForTxnTopic(): Unit = {
testFindCoordinatorWithTopicCreation(CoordinatorType.TRANSACTION)
}
@Test