From 807a6e2d2d763dba01b42490b02c5d789cffb04d Mon Sep 17 00:00:00 2001 From: Sanskar Jhajharia Date: Thu, 23 May 2024 21:40:58 +0530 Subject: [PATCH 1/3] Clients Code Cleanup --- .../apache/kafka/clients/NodeApiVersions.java | 7 +- .../apache/kafka/clients/admin/NewTopic.java | 14 +- .../consumer/internals/MemberState.java | 7 +- .../producer/internals/RecordAccumulator.java | 13 +- .../record/AbstractLegacyRecordBatch.java | 2 +- .../requests/DescribeLogDirsResponse.java | 28 ++- .../kafka/common/requests/FetchRequest.java | 26 ++- .../common/requests/LeaderAndIsrRequest.java | 18 +- .../kafka/common/requests/ProduceRequest.java | 14 +- .../common/requests/StopReplicaRequest.java | 16 +- .../requests/UpdateMetadataRequest.java | 20 +-- ...shingHttpsJwksVerificationKeyResolver.java | 18 +- .../unsecured/OAuthBearerValidationUtils.java | 4 +- .../security/ssl/DefaultSslEngineFactory.java | 5 +- .../security/ssl/SslPrincipalMapper.java | 2 +- .../org/apache/kafka/common/utils/Utils.java | 1 + .../apache/kafka/clients/ClientUtilsTest.java | 2 +- .../apache/kafka/clients/MetadataTest.java | 22 +-- .../kafka/clients/admin/ConfigTest.java | 5 +- ...escribeUserScramCredentialsResultTest.java | 13 +- .../clients/admin/KafkaAdminClientTest.java | 165 +++++++++--------- .../kafka/clients/admin/MockAdminClient.java | 2 +- .../internals/CoordinatorStrategyTest.java | 2 +- .../internals/CommitRequestManagerTest.java | 29 ++- .../internals/ConsumerCoordinatorTest.java | 26 +-- .../internals/ConsumerInterceptorsTest.java | 2 +- .../clients/producer/KafkaProducerTest.java | 2 +- .../internals/RecordAccumulatorTest.java | 14 +- .../internals/TransactionManagerTest.java | 4 +- .../kafka/common/config/ConfigDefTest.java | 20 +-- .../provider/DirectoryConfigProviderTest.java | 4 +- .../kafka/common/metrics/JmxReporterTest.java | 6 +- .../kafka/common/metrics/MetricsTest.java | 10 +- .../kafka/common/metrics/SensorTest.java | 3 +- .../common/metrics/stats/FrequenciesTest.java | 3 +- .../kafka/common/network/SelectorTest.java | 2 +- .../common/network/SslTransportLayerTest.java | 9 +- .../network/SslTransportTls12Tls13Test.java | 2 +- .../common/network/Tls12SelectorTest.java | 4 +- .../common/network/Tls13SelectorTest.java | 3 +- .../common/protocol/MessageUtilTest.java | 2 +- .../record/MemoryRecordsBuilderTest.java | 2 +- .../AlterReplicaLogDirsRequestTest.java | 6 +- .../AlterReplicaLogDirsResponseTest.java | 3 +- .../requests/DeleteAclsResponseTest.java | 2 +- .../requests/LeaderAndIsrRequestTest.java | 4 +- .../requests/ListOffsetsRequestTest.java | 20 +-- .../common/requests/RequestResponseTest.java | 32 ++-- .../requests/UpdateMetadataRequestTest.java | 14 +- .../ClientAuthenticationFailureTest.java | 5 +- .../SaslAuthenticatorFailureDelayTest.java | 5 +- .../authenticator/SaslAuthenticatorTest.java | 147 ++++++++-------- .../security/kerberos/KerberosNameTest.java | 25 +-- ...arerUnsecuredLoginCallbackHandlerTest.java | 2 +- ...UnsecuredValidatorCallbackHandlerTest.java | 3 +- .../OAuthBearerValidationUtilsTest.java | 6 +- .../common/utils/FlattenedIteratorTest.java | 21 +-- 57 files changed, 409 insertions(+), 439 deletions(-) diff --git a/clients/src/main/java/org/apache/kafka/clients/NodeApiVersions.java b/clients/src/main/java/org/apache/kafka/clients/NodeApiVersions.java index 838718652f37c..12a7d019921b5 100644 --- a/clients/src/main/java/org/apache/kafka/clients/NodeApiVersions.java +++ b/clients/src/main/java/org/apache/kafka/clients/NodeApiVersions.java @@ -174,10 +174,9 @@ public String toString(boolean lineBreaks) { // which may happen when the remote is too old. for (ApiKeys apiKey : ApiKeys.clientApis()) { if (!apiKeysText.containsKey(apiKey.id)) { - StringBuilder bld = new StringBuilder(); - bld.append(apiKey.name).append("("). - append(apiKey.id).append("): ").append("UNSUPPORTED"); - apiKeysText.put(apiKey.id, bld.toString()); + String bld = apiKey.name + "(" + + apiKey.id + "): " + "UNSUPPORTED"; + apiKeysText.put(apiKey.id, bld); } } String separator = lineBreaks ? ",\n\t" : ", "; diff --git a/clients/src/main/java/org/apache/kafka/clients/admin/NewTopic.java b/clients/src/main/java/org/apache/kafka/clients/admin/NewTopic.java index 2f335d02f2f2b..0151e6f61793f 100644 --- a/clients/src/main/java/org/apache/kafka/clients/admin/NewTopic.java +++ b/clients/src/main/java/org/apache/kafka/clients/admin/NewTopic.java @@ -147,14 +147,12 @@ CreatableTopic convertToCreatableTopic() { @Override public String toString() { - StringBuilder bld = new StringBuilder(); - bld.append("(name=").append(name). - append(", numPartitions=").append(numPartitions.map(String::valueOf).orElse("default")). - append(", replicationFactor=").append(replicationFactor.map(String::valueOf).orElse("default")). - append(", replicasAssignments=").append(replicasAssignments). - append(", configs=").append(configs). - append(")"); - return bld.toString(); + return "(name=" + name + + ", numPartitions=" + numPartitions.map(String::valueOf).orElse("default") + + ", replicationFactor=" + replicationFactor.map(String::valueOf).orElse("default") + + ", replicasAssignments=" + replicasAssignments + + ", configs=" + configs + + ")"; } @Override diff --git a/clients/src/main/java/org/apache/kafka/clients/consumer/internals/MemberState.java b/clients/src/main/java/org/apache/kafka/clients/consumer/internals/MemberState.java index 9f0c7d947ea7e..a62c634ba437e 100644 --- a/clients/src/main/java/org/apache/kafka/clients/consumer/internals/MemberState.java +++ b/clients/src/main/java/org/apache/kafka/clients/consumer/internals/MemberState.java @@ -21,6 +21,7 @@ import java.util.ArrayList; import java.util.Arrays; +import java.util.Collections; import java.util.List; public enum MemberState { @@ -120,7 +121,7 @@ public enum MemberState { RECONCILING.previousValidStates = Arrays.asList(STABLE, JOINING, ACKNOWLEDGING, RECONCILING); - ACKNOWLEDGING.previousValidStates = Arrays.asList(RECONCILING); + ACKNOWLEDGING.previousValidStates = Collections.singletonList(RECONCILING); FATAL.previousValidStates = Arrays.asList(JOINING, STABLE, RECONCILING, ACKNOWLEDGING, PREPARE_LEAVING, LEAVING, UNSUBSCRIBED); @@ -133,11 +134,11 @@ public enum MemberState { PREPARE_LEAVING.previousValidStates = Arrays.asList(JOINING, STABLE, RECONCILING, ACKNOWLEDGING, UNSUBSCRIBED); - LEAVING.previousValidStates = Arrays.asList(PREPARE_LEAVING); + LEAVING.previousValidStates = Collections.singletonList(PREPARE_LEAVING); UNSUBSCRIBED.previousValidStates = Arrays.asList(PREPARE_LEAVING, LEAVING, FENCED); - STALE.previousValidStates = Arrays.asList(LEAVING); + STALE.previousValidStates = Collections.singletonList(LEAVING); } private List previousValidStates; diff --git a/clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java b/clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java index 12e77b0d516b7..9ea7a32f949c5 100644 --- a/clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java +++ b/clients/src/main/java/org/apache/kafka/clients/producer/internals/RecordAccumulator.java @@ -855,13 +855,12 @@ private boolean shouldStopDrainBatchesForPartition(ProducerBatch first, TopicPar } int firstInFlightSequence = transactionManager.firstInFlightSequence(first.topicPartition); - if (firstInFlightSequence != RecordBatch.NO_SEQUENCE && first.hasSequence() - && first.baseSequence() != firstInFlightSequence) - // If the queued batch already has an assigned sequence, then it is being retried. - // In this case, we wait until the next immediate batch is ready and drain that. - // We only move on when the next in line batch is complete (either successfully or due to - // a fatal broker error). This effectively reduces our in flight request count to 1. - return true; + // If the queued batch already has an assigned sequence, then it is being retried. + // In this case, we wait until the next immediate batch is ready and drain that. + // We only move on when the next in line batch is complete (either successfully or due to + // a fatal broker error). This effectively reduces our in flight request count to 1. + return firstInFlightSequence != RecordBatch.NO_SEQUENCE && first.hasSequence() + && first.baseSequence() != firstInFlightSequence; } return false; } diff --git a/clients/src/main/java/org/apache/kafka/common/record/AbstractLegacyRecordBatch.java b/clients/src/main/java/org/apache/kafka/common/record/AbstractLegacyRecordBatch.java index 0f29581623754..9ab8715236e74 100644 --- a/clients/src/main/java/org/apache/kafka/common/record/AbstractLegacyRecordBatch.java +++ b/clients/src/main/java/org/apache/kafka/common/record/AbstractLegacyRecordBatch.java @@ -279,7 +279,7 @@ static void writeHeader(DataOutputStream out, long offset, int size) throws IOEx private static final class DataLogInputStream implements LogInputStream { private final InputStream stream; - protected final int maxMessageSize; + private final int maxMessageSize; private final ByteBuffer offsetAndSizeBuffer; DataLogInputStream(InputStream stream, int maxMessageSize) { diff --git a/clients/src/main/java/org/apache/kafka/common/requests/DescribeLogDirsResponse.java b/clients/src/main/java/org/apache/kafka/common/requests/DescribeLogDirsResponse.java index cbf3054217363..3d06cff712c18 100644 --- a/clients/src/main/java/org/apache/kafka/common/requests/DescribeLogDirsResponse.java +++ b/clients/src/main/java/org/apache/kafka/common/requests/DescribeLogDirsResponse.java @@ -93,13 +93,11 @@ public LogDirInfo(Errors error, Map replicaInfos) { @Override public String toString() { - StringBuilder builder = new StringBuilder(); - builder.append("(error=") - .append(error) - .append(", replicas=") - .append(replicaInfos) - .append(")"); - return builder.toString(); + return "(error=" + + error + + ", replicas=" + + replicaInfos + + ")"; } } @@ -126,15 +124,13 @@ public ReplicaInfo(long size, long offsetLag, boolean isFuture) { @Override public String toString() { - StringBuilder builder = new StringBuilder(); - builder.append("(size=") - .append(size) - .append(", offsetLag=") - .append(offsetLag) - .append(", isFuture=") - .append(isFuture) - .append(")"); - return builder.toString(); + return "(size=" + + size + + ", offsetLag=" + + offsetLag + + ", isFuture=" + + isFuture + + ")"; } } diff --git a/clients/src/main/java/org/apache/kafka/common/requests/FetchRequest.java b/clients/src/main/java/org/apache/kafka/common/requests/FetchRequest.java index 2065a15d94259..1082200ec393c 100644 --- a/clients/src/main/java/org/apache/kafka/common/requests/FetchRequest.java +++ b/clients/src/main/java/org/apache/kafka/common/requests/FetchRequest.java @@ -316,20 +316,18 @@ public FetchRequest build(short version) { @Override public String toString() { - StringBuilder bld = new StringBuilder(); - bld.append("(type=FetchRequest"). - append(", replicaId=").append(replicaId). - append(", maxWait=").append(maxWait). - append(", minBytes=").append(minBytes). - append(", maxBytes=").append(maxBytes). - append(", fetchData=").append(toFetch). - append(", isolationLevel=").append(isolationLevel). - append(", removed=").append(removed.stream().map(TopicIdPartition::toString).collect(Collectors.joining(", "))). - append(", replaced=").append(replaced.stream().map(TopicIdPartition::toString).collect(Collectors.joining(", "))). - append(", metadata=").append(metadata). - append(", rackId=").append(rackId). - append(")"); - return bld.toString(); + return "(type=FetchRequest" + + ", replicaId=" + replicaId + + ", maxWait=" + maxWait + + ", minBytes=" + minBytes + + ", maxBytes=" + maxBytes + + ", fetchData=" + toFetch + + ", isolationLevel=" + isolationLevel + + ", removed=" + removed.stream().map(TopicIdPartition::toString).collect(Collectors.joining(", ")) + + ", replaced=" + replaced.stream().map(TopicIdPartition::toString).collect(Collectors.joining(", ")) + + ", metadata=" + metadata + + ", rackId=" + rackId + + ")"; } } diff --git a/clients/src/main/java/org/apache/kafka/common/requests/LeaderAndIsrRequest.java b/clients/src/main/java/org/apache/kafka/common/requests/LeaderAndIsrRequest.java index 8caddb0054169..9fc83cfd847a8 100644 --- a/clients/src/main/java/org/apache/kafka/common/requests/LeaderAndIsrRequest.java +++ b/clients/src/main/java/org/apache/kafka/common/requests/LeaderAndIsrRequest.java @@ -112,16 +112,14 @@ private static Map groupByTopic(List d.partitionData().stream()).collect(Collectors.toList())) - .append("), transactionalId='").append(data.transactionalId() != null ? data.transactionalId() : "") - .append("'"); - return bld.toString(); + return "(type=ProduceRequest" + + ", acks=" + data.acks() + + ", timeout=" + data.timeoutMs() + + ", partitionRecords=(" + data.topicData().stream().flatMap(d -> d.partitionData().stream()).collect(Collectors.toList()) + + "), transactionalId='" + (data.transactionalId() != null ? data.transactionalId() : "") + + "'"; } } diff --git a/clients/src/main/java/org/apache/kafka/common/requests/StopReplicaRequest.java b/clients/src/main/java/org/apache/kafka/common/requests/StopReplicaRequest.java index 940a16f0a8589..6245cb27c6c47 100644 --- a/clients/src/main/java/org/apache/kafka/common/requests/StopReplicaRequest.java +++ b/clients/src/main/java/org/apache/kafka/common/requests/StopReplicaRequest.java @@ -94,15 +94,13 @@ public StopReplicaRequest build(short version) { @Override public String toString() { - StringBuilder bld = new StringBuilder(); - bld.append("(type=StopReplicaRequest"). - append(", controllerId=").append(controllerId). - append(", controllerEpoch=").append(controllerEpoch). - append(", brokerEpoch=").append(brokerEpoch). - append(", deletePartitions=").append(deletePartitions). - append(", topicStates=").append(topicStates.stream().map(StopReplicaTopicState::toString).collect(Collectors.joining(","))). - append(")"); - return bld.toString(); + return "(type=StopReplicaRequest" + + ", controllerId=" + controllerId + + ", controllerEpoch=" + controllerEpoch + + ", brokerEpoch=" + brokerEpoch + + ", deletePartitions=" + deletePartitions + + ", topicStates=" + topicStates.stream().map(StopReplicaTopicState::toString).collect(Collectors.joining(",")) + + ")"; } } diff --git a/clients/src/main/java/org/apache/kafka/common/requests/UpdateMetadataRequest.java b/clients/src/main/java/org/apache/kafka/common/requests/UpdateMetadataRequest.java index b846fb7b0f9ed..15a4dfff1a6a0 100644 --- a/clients/src/main/java/org/apache/kafka/common/requests/UpdateMetadataRequest.java +++ b/clients/src/main/java/org/apache/kafka/common/requests/UpdateMetadataRequest.java @@ -133,17 +133,15 @@ private static Map groupByTopic(Map nestingContex if (refreshingHttpsJwks.maybeExpediteRefresh(keyId)) log.debug("Refreshing JWKs from {} as no suitable verification key for JWS w/ header {} was found in {}", refreshingHttpsJwks.getLocation(), jws.getHeaders().getFullHeaderAsJsonString(), jwks); - StringBuilder sb = new StringBuilder(); - sb.append("Unable to find a suitable verification key for JWS w/ header ").append(jws.getHeaders().getFullHeaderAsJsonString()); - sb.append(" from JWKs ").append(jwks).append(" obtained from ").append( - refreshingHttpsJwks.getLocation()); - throw new UnresolvableKeyException(sb.toString()); + String sb = "Unable to find a suitable verification key for JWS w/ header " + jws.getHeaders().getFullHeaderAsJsonString() + + " from JWKs " + jwks + " obtained from " + + refreshingHttpsJwks.getLocation(); + throw new UnresolvableKeyException(sb); } catch (JoseException | IOException e) { - StringBuilder sb = new StringBuilder(); - sb.append("Unable to find a suitable verification key for JWS w/ header ").append(jws.getHeaders().getFullHeaderAsJsonString()); - sb.append(" due to an unexpected exception (").append(e).append(") while obtaining or using keys from JWKS endpoint at ").append( - refreshingHttpsJwks.getLocation()); - throw new UnresolvableKeyException(sb.toString(), e); + String sb = "Unable to find a suitable verification key for JWS w/ header " + jws.getHeaders().getFullHeaderAsJsonString() + + " due to an unexpected exception (" + e + ") while obtaining or using keys from JWKS endpoint at " + + refreshingHttpsJwks.getLocation(); + throw new UnresolvableKeyException(sb, e); } } diff --git a/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java b/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java index f12a482f14901..c9eaac41b4545 100644 --- a/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java +++ b/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java @@ -176,8 +176,8 @@ public static OAuthBearerValidationResult validateScope(OAuthBearerToken token, if (!tokenScope.contains(requiredScopeElement)) return OAuthBearerValidationResult.newFailure(String.format( "The provided scope (%s) was missing a required scope (%s). All required scope elements: %s", - String.valueOf(tokenScope), requiredScopeElement, requiredScope), - requiredScope.toString(), null); + tokenScope, requiredScopeElement, requiredScope), + requiredScope.toString(), null); } return OAuthBearerValidationResult.newSuccess(); } diff --git a/clients/src/main/java/org/apache/kafka/common/security/ssl/DefaultSslEngineFactory.java b/clients/src/main/java/org/apache/kafka/common/security/ssl/DefaultSslEngineFactory.java index 3ca8ca6fcc027..3bdf9b70f2a0a 100644 --- a/clients/src/main/java/org/apache/kafka/common/security/ssl/DefaultSslEngineFactory.java +++ b/clients/src/main/java/org/apache/kafka/common/security/ssl/DefaultSslEngineFactory.java @@ -109,10 +109,7 @@ public boolean shouldBeRebuilt(Map nextConfigs) { if (truststore != null && truststore.modified()) { return true; } - if (keystore != null && keystore.modified()) { - return true; - } - return false; + return keystore != null && keystore.modified(); } @Override diff --git a/clients/src/main/java/org/apache/kafka/common/security/ssl/SslPrincipalMapper.java b/clients/src/main/java/org/apache/kafka/common/security/ssl/SslPrincipalMapper.java index 0fb83281bf864..bd8c50a0a8b7f 100644 --- a/clients/src/main/java/org/apache/kafka/common/security/ssl/SslPrincipalMapper.java +++ b/clients/src/main/java/org/apache/kafka/common/security/ssl/SslPrincipalMapper.java @@ -178,7 +178,7 @@ private String escapeLiteralBackReferences(final String unescaped, final int num final StringBuilder sb = new StringBuilder(value.length() + 1); final int groupStart = backRefMatcher.start(1); - sb.append(value.substring(0, groupStart - 1)); + sb.append(value, 0, groupStart - 1); sb.append("\\"); sb.append(value.substring(groupStart - 1)); value = sb.toString(); diff --git a/clients/src/main/java/org/apache/kafka/common/utils/Utils.java b/clients/src/main/java/org/apache/kafka/common/utils/Utils.java index ce67fbdb0c73e..df7f5a50c1dce 100644 --- a/clients/src/main/java/org/apache/kafka/common/utils/Utils.java +++ b/clients/src/main/java/org/apache/kafka/common/utils/Utils.java @@ -371,6 +371,7 @@ public static boolean isEqualConstantTime(char[] first, char[] second) { int j = i < second.length ? i : 0; if (first[i] != second[j]) { matches = false; + break; } } return matches; diff --git a/clients/src/test/java/org/apache/kafka/clients/ClientUtilsTest.java b/clients/src/test/java/org/apache/kafka/clients/ClientUtilsTest.java index 80e5ddfc4d830..536a27f576fd0 100644 --- a/clients/src/test/java/org/apache/kafka/clients/ClientUtilsTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/ClientUtilsTest.java @@ -57,7 +57,7 @@ public void testParseAndValidateAddressesWithReverseLookup() { // With lookup of example.com, either one or two addresses are expected depending on // whether ipv4 and ipv6 are enabled - List validatedAddresses = checkWithLookup(asList("example.com:10000")); + List validatedAddresses = checkWithLookup(Collections.singletonList("example.com:10000")); assertFalse(validatedAddresses.isEmpty(), "Unexpected addresses " + validatedAddresses); List validatedHostNames = validatedAddresses.stream().map(InetSocketAddress::getHostName) .collect(Collectors.toList()); diff --git a/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java b/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java index 0b2733207ce42..d25468cb03018 100644 --- a/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java @@ -1135,13 +1135,13 @@ public void testTopicMetadataOnUpdatePartitionLeadership() { // "input-topic" partition 1 leader changes from node 1 to node 2 metadata.updatePartitionLeadership( - Collections.singletonMap( - tp1, - new Metadata.LeaderIdAndEpoch( - Optional.of(2), - Optional.of(3) - )), - Arrays.asList(node1) + Collections.singletonMap( + tp1, + new Metadata.LeaderIdAndEpoch( + Optional.of(2), + Optional.of(3) + )), + Collections.singletonList(node1) ); assertEquals(2, metadata.fetch().partitionsForTopic(topic).size()); assertEquals(1, metadata.fetch().partition(tp0).leader().id()); @@ -1161,20 +1161,20 @@ public void testUpdatePartitionLeadership() { // topic2 has 1 partition: tp21 String topic1 = "topic1"; TopicPartition tp11 = new TopicPartition(topic1, 0); - PartitionMetadata part1Metadata = new PartitionMetadata(Errors.NONE, tp11, Optional.of(1), Optional.of(100), Arrays.asList(1, 2), Arrays.asList(1, 2), Arrays.asList(3)); + PartitionMetadata part1Metadata = new PartitionMetadata(Errors.NONE, tp11, Optional.of(1), Optional.of(100), Arrays.asList(1, 2), Arrays.asList(1, 2), Collections.singletonList(3)); Uuid topic1Id = Uuid.randomUuid(); TopicPartition tp12 = new TopicPartition(topic1, 1); - PartitionMetadata part12Metadata = new PartitionMetadata(Errors.NONE, tp12, Optional.of(2), Optional.of(200), Arrays.asList(2, 3), Arrays.asList(2, 3), Arrays.asList(1)); + PartitionMetadata part12Metadata = new PartitionMetadata(Errors.NONE, tp12, Optional.of(2), Optional.of(200), Arrays.asList(2, 3), Arrays.asList(2, 3), Collections.singletonList(1)); String topic2 = "topic2"; TopicPartition tp21 = new TopicPartition(topic2, 0); - PartitionMetadata part2Metadata = new PartitionMetadata(Errors.NONE, tp21, Optional.of(2), Optional.of(200), Arrays.asList(2, 3), Arrays.asList(2, 3), Arrays.asList(1)); + PartitionMetadata part2Metadata = new PartitionMetadata(Errors.NONE, tp21, Optional.of(2), Optional.of(200), Arrays.asList(2, 3), Arrays.asList(2, 3), Collections.singletonList(1)); Uuid topic2Id = Uuid.randomUuid(); Set internalTopics = Collections.singleton(Topic.GROUP_METADATA_TOPIC_NAME); TopicPartition internalPart = new TopicPartition(Topic.GROUP_METADATA_TOPIC_NAME, 0); Uuid internalTopicId = Uuid.randomUuid(); - PartitionMetadata internalTopicMetadata = new PartitionMetadata(Errors.NONE, internalPart, Optional.of(2), Optional.of(200), Arrays.asList(2, 3), Arrays.asList(2, 3), Arrays.asList(1)); + PartitionMetadata internalTopicMetadata = new PartitionMetadata(Errors.NONE, internalPart, Optional.of(2), Optional.of(200), Arrays.asList(2, 3), Arrays.asList(2, 3), Collections.singletonList(1)); Map topicIds = new HashMap<>(); topicIds.put(topic1, topic1Id); diff --git a/clients/src/test/java/org/apache/kafka/clients/admin/ConfigTest.java b/clients/src/test/java/org/apache/kafka/clients/admin/ConfigTest.java index 59d1150ac3ba8..7d70d58a71ac2 100644 --- a/clients/src/test/java/org/apache/kafka/clients/admin/ConfigTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/admin/ConfigTest.java @@ -21,6 +21,7 @@ import org.junit.jupiter.api.BeforeEach; import org.junit.jupiter.api.Test; +import java.util.Collections; import java.util.List; import static java.util.Arrays.asList; @@ -61,7 +62,7 @@ public void shouldGetAllEntries() { public void shouldImplementEqualsProperly() { assertEquals(config, config); assertEquals(config, new Config(config.entries())); - assertNotEquals(new Config(asList(E1)), config); + assertNotEquals(new Config(Collections.singletonList(E1)), config); assertNotEquals(config, "this"); } @@ -69,7 +70,7 @@ public void shouldImplementEqualsProperly() { public void shouldImplementHashCodeProperly() { assertEquals(config.hashCode(), config.hashCode()); assertEquals(config.hashCode(), new Config(config.entries()).hashCode()); - assertNotEquals(new Config(asList(E1)).hashCode(), config.hashCode()); + assertNotEquals(new Config(Collections.singletonList(E1)).hashCode(), config.hashCode()); } @Test diff --git a/clients/src/test/java/org/apache/kafka/clients/admin/DescribeUserScramCredentialsResultTest.java b/clients/src/test/java/org/apache/kafka/clients/admin/DescribeUserScramCredentialsResultTest.java index 9b5e98a005f06..13119e9f2cf5e 100644 --- a/clients/src/test/java/org/apache/kafka/clients/admin/DescribeUserScramCredentialsResultTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/admin/DescribeUserScramCredentialsResultTest.java @@ -23,6 +23,7 @@ import org.junit.jupiter.api.Test; import java.util.Arrays; +import java.util.Collections; import java.util.Map; import static org.junit.jupiter.api.Assertions.assertEquals; @@ -64,7 +65,7 @@ public void testUserLevelErrors() throws Exception { int iterations = 4096; dataFuture.complete(new DescribeUserScramCredentialsResponseData().setErrorCode(Errors.NONE.code()).setResults(Arrays.asList( new DescribeUserScramCredentialsResponseData.DescribeUserScramCredentialsResult().setUser(goodUser).setCredentialInfos( - Arrays.asList(new DescribeUserScramCredentialsResponseData.CredentialInfo().setMechanism(scramSha256.type()).setIterations(iterations))), + Collections.singletonList(new DescribeUserScramCredentialsResponseData.CredentialInfo().setMechanism(scramSha256.type()).setIterations(iterations))), new DescribeUserScramCredentialsResponseData.DescribeUserScramCredentialsResult().setUser(unknownUser).setErrorCode(Errors.RESOURCE_NOT_FOUND.code()), new DescribeUserScramCredentialsResponseData.DescribeUserScramCredentialsResult().setUser(failedUser).setErrorCode(Errors.DUPLICATE_RESOURCE.code())))); DescribeUserScramCredentialsResult results = new DescribeUserScramCredentialsResult(dataFuture); @@ -76,7 +77,7 @@ public void testUserLevelErrors() throws Exception { } assertEquals(Arrays.asList(goodUser, failedUser), results.users().get(), "Expected 2 users with credentials"); UserScramCredentialsDescription goodUserDescription = results.description(goodUser).get(); - assertEquals(new UserScramCredentialsDescription(goodUser, Arrays.asList(new ScramCredentialInfo(scramSha256, iterations))), goodUserDescription); + assertEquals(new UserScramCredentialsDescription(goodUser, Collections.singletonList(new ScramCredentialInfo(scramSha256, iterations))), goodUserDescription); try { results.description(failedUser).get(); fail("expected description(failedUser) to fail when there is a user-level error"); @@ -98,15 +99,15 @@ public void testSuccessfulDescription() throws Exception { KafkaFutureImpl dataFuture = new KafkaFutureImpl<>(); ScramMechanism scramSha256 = ScramMechanism.SCRAM_SHA_256; int iterations = 4096; - dataFuture.complete(new DescribeUserScramCredentialsResponseData().setErrorCode(Errors.NONE.code()).setResults(Arrays.asList( + dataFuture.complete(new DescribeUserScramCredentialsResponseData().setErrorCode(Errors.NONE.code()).setResults(Collections.singletonList( new DescribeUserScramCredentialsResponseData.DescribeUserScramCredentialsResult().setUser(goodUser).setCredentialInfos( - Arrays.asList(new DescribeUserScramCredentialsResponseData.CredentialInfo().setMechanism(scramSha256.type()).setIterations(iterations)))))); + Collections.singletonList(new DescribeUserScramCredentialsResponseData.CredentialInfo().setMechanism(scramSha256.type()).setIterations(iterations)))))); DescribeUserScramCredentialsResult results = new DescribeUserScramCredentialsResult(dataFuture); - assertEquals(Arrays.asList(goodUser), results.users().get(), "Expected 1 user with credentials"); + assertEquals(Collections.singletonList(goodUser), results.users().get(), "Expected 1 user with credentials"); Map allResults = results.all().get(); assertEquals(1, allResults.size()); UserScramCredentialsDescription goodUserDescriptionViaAll = allResults.get(goodUser); - assertEquals(new UserScramCredentialsDescription(goodUser, Arrays.asList(new ScramCredentialInfo(scramSha256, iterations))), goodUserDescriptionViaAll); + assertEquals(new UserScramCredentialsDescription(goodUser, Collections.singletonList(new ScramCredentialInfo(scramSha256, iterations))), goodUserDescriptionViaAll); assertEquals(goodUserDescriptionViaAll, results.description(goodUser).get(), "Expected same thing via all() and description()"); try { results.description(unknownUser).get(); diff --git a/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java b/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java index ea1305e533bef..eb7aea8ae871d 100644 --- a/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java @@ -1426,7 +1426,7 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiBasic() { ); DescribeTopicPartitionsResponseData dataFirstPart = new DescribeTopicPartitionsResponseData(); - addPartitionToDescribeTopicPartitionsResponse(dataFirstPart, topicName0, topics.get(topicName0), Arrays.asList(0)); + addPartitionToDescribeTopicPartitionsResponse(dataFirstPart, topicName0, topics.get(topicName0), singletonList(0)); dataFirstPart.setNextCursor(new DescribeTopicPartitionsResponseData.Cursor() .setTopicName(topicName0) .setPartitionIndex(1)); @@ -1435,13 +1435,12 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiBasic() { if (request.topics().size() != 2) return false; if (!request.topics().get(0).name().equals(topicName0)) return false; if (!request.topics().get(1).name().equals(topicName1)) return false; - if (request.cursor() != null) return false; - return true; + return request.cursor() == null; }, new DescribeTopicPartitionsResponse(dataFirstPart)); DescribeTopicPartitionsResponseData dataSecondPart = new DescribeTopicPartitionsResponseData(); - addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName0, topics.get(topicName0), Arrays.asList(1)); - addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName1, topics.get(topicName1), Arrays.asList(0)); + addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName0, topics.get(topicName0), singletonList(1)); + addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName1, topics.get(topicName1), singletonList(0)); env.kafkaClient().prepareResponse(body -> { DescribeTopicPartitionsRequestData request = (DescribeTopicPartitionsRequestData) body.data(); if (request.topics().size() != 2) return false; @@ -1449,9 +1448,7 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiBasic() { if (!request.topics().get(1).name().equals(topicName1)) return false; DescribeTopicPartitionsRequestData.Cursor cursor = request.cursor(); - if (cursor == null || cursor.topicName() != topicName0 || cursor.partitionIndex() != 1) return false; - - return true; + return cursor != null && cursor.topicName() == topicName0 && cursor.partitionIndex() == 1; }, new DescribeTopicPartitionsResponse(dataSecondPart)); try { DescribeTopicsResult result = env.adminClient().describeTopics( @@ -1493,8 +1490,8 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiEdgeCase() { ); DescribeTopicPartitionsResponseData dataFirstPart = new DescribeTopicPartitionsResponseData(); - addPartitionToDescribeTopicPartitionsResponse(dataFirstPart, topicName0, topics.get(topicName0), Arrays.asList(0)); - addPartitionToDescribeTopicPartitionsResponse(dataFirstPart, topicName1, topics.get(topicName1), Arrays.asList(0)); + addPartitionToDescribeTopicPartitionsResponse(dataFirstPart, topicName0, topics.get(topicName0), singletonList(0)); + addPartitionToDescribeTopicPartitionsResponse(dataFirstPart, topicName1, topics.get(topicName1), singletonList(0)); dataFirstPart.setNextCursor(new DescribeTopicPartitionsResponseData.Cursor() .setTopicName(topicName1) .setPartitionIndex(1)); @@ -1504,13 +1501,12 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiEdgeCase() { if (!request.topics().get(0).name().equals(topicName0)) return false; if (!request.topics().get(1).name().equals(topicName1)) return false; if (!request.topics().get(2).name().equals(topicName2)) return false; - if (request.cursor() != null) return false; - return true; + return request.cursor() == null; }, new DescribeTopicPartitionsResponse(dataFirstPart)); DescribeTopicPartitionsResponseData dataSecondPart = new DescribeTopicPartitionsResponseData(); - addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName1, topics.get(topicName1), Arrays.asList(1)); - addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName2, topics.get(topicName2), Arrays.asList(0)); + addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName1, topics.get(topicName1), singletonList(1)); + addPartitionToDescribeTopicPartitionsResponse(dataSecondPart, topicName2, topics.get(topicName2), singletonList(0)); dataSecondPart.setNextCursor(new DescribeTopicPartitionsResponseData.Cursor() .setTopicName(topicName2) .setPartitionIndex(1)); @@ -1520,19 +1516,17 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiEdgeCase() { if (!request.topics().get(0).name().equals(topicName1)) return false; if (!request.topics().get(1).name().equals(topicName2)) return false; DescribeTopicPartitionsRequestData.Cursor cursor = request.cursor(); - if (cursor == null || !cursor.topicName().equals(topicName1) || cursor.partitionIndex() != 1) return false; - return true; + return cursor != null && cursor.topicName().equals(topicName1) && cursor.partitionIndex() == 1; }, new DescribeTopicPartitionsResponse(dataSecondPart)); DescribeTopicPartitionsResponseData dataThirdPart = new DescribeTopicPartitionsResponseData(); - addPartitionToDescribeTopicPartitionsResponse(dataThirdPart, topicName2, topics.get(topicName2), Arrays.asList(1)); + addPartitionToDescribeTopicPartitionsResponse(dataThirdPart, topicName2, topics.get(topicName2), singletonList(1)); env.kafkaClient().prepareResponse(body -> { DescribeTopicPartitionsRequestData request = (DescribeTopicPartitionsRequestData) body.data(); if (request.topics().size() != 1) return false; if (!request.topics().get(0).name().equals(topicName2)) return false; DescribeTopicPartitionsRequestData.Cursor cursor = request.cursor(); - if (cursor == null || !cursor.topicName().equals(topicName2) || cursor.partitionIndex() != 1) return false; - return true; + return cursor != null && cursor.topicName().equals(topicName2) && cursor.partitionIndex() == 1; }, new DescribeTopicPartitionsResponse(dataThirdPart)); try { DescribeTopicsResult result = env.adminClient().describeTopics( @@ -1561,12 +1555,12 @@ private void addPartitionToDescribeTopicPartitionsResponse( List addingPartitions = new ArrayList<>(); partitions.forEach(partition -> { addingPartitions.add(new DescribeTopicPartitionsResponsePartition() - .setIsrNodes(Arrays.asList(0)) + .setIsrNodes(singletonList(0)) .setErrorCode((short) 0) .setLeaderEpoch(0) .setLeaderId(0) - .setEligibleLeaderReplicas(Arrays.asList(1)) - .setLastKnownElr(Arrays.asList(2)) + .setEligibleLeaderReplicas(singletonList(1)) + .setLastKnownElr(singletonList(2)) .setPartitionIndex(partition) .setReplicaNodes(Arrays.asList(0, 1, 2))); }); @@ -1603,15 +1597,15 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiErrorHandling() { .setTopicId(topics.get(topicName0)) .setName(topicName0) .setIsInternal(false) - .setPartitions(Arrays.asList(new DescribeTopicPartitionsResponsePartition() - .setIsrNodes(Arrays.asList(0)) - .setErrorCode((short) 0) - .setLeaderEpoch(0) - .setLeaderId(0) - .setEligibleLeaderReplicas(Arrays.asList(1)) - .setLastKnownElr(Arrays.asList(2)) - .setPartitionIndex(0) - .setReplicaNodes(Arrays.asList(0, 1, 2)))) + .setPartitions(singletonList(new DescribeTopicPartitionsResponsePartition() + .setIsrNodes(singletonList(0)) + .setErrorCode((short) 0) + .setLeaderEpoch(0) + .setLeaderId(0) + .setEligibleLeaderReplicas(singletonList(1)) + .setLastKnownElr(singletonList(2)) + .setPartitionIndex(0) + .setReplicaNodes(asList(0, 1, 2)))) ); dataFirstPart.topics().add(new DescribeTopicPartitionsResponseTopic() .setErrorCode((short) 29) @@ -1624,8 +1618,7 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiErrorHandling() { if (request.topics().size() != 2) return false; if (!request.topics().get(0).name().equals(topicName0)) return false; if (!request.topics().get(1).name().equals(topicName1)) return false; - if (request.cursor() != null) return false; - return true; + return request.cursor() == null; }, new DescribeTopicPartitionsResponse(dataFirstPart)); DescribeTopicsResult result = env.adminClient().describeTopics( Arrays.asList(topicName1, topicName0), new DescribeTopicsOptions() @@ -1660,7 +1653,7 @@ private void callAdminClientApisAndExpectAnAuthenticationError(AdminClientUnitTe Map counts = new HashMap<>(); counts.put("my_topic", NewPartitions.increaseTo(3)); - counts.put("other_topic", NewPartitions.increaseTo(3, asList(asList(2), asList(3)))); + counts.put("other_topic", NewPartitions.increaseTo(3, asList(singletonList(2), singletonList(3)))); e = assertThrows(ExecutionException.class, () -> env.adminClient().createPartitions(counts).all().get()); assertInstanceOf(AuthenticationException.class, e.getCause(), "Expected an authentication error, but got " + Utils.stackTrace(e)); @@ -1690,9 +1683,9 @@ private void callClientQuotasApisAndExpectAnAuthenticationError(AdminClientUnitT "Expected an authentication error, but got " + Utils.stackTrace(e)); ClientQuotaEntity entity = new ClientQuotaEntity(Collections.singletonMap(ClientQuotaEntity.USER, "user")); - ClientQuotaAlteration alteration = new ClientQuotaAlteration(entity, asList(new ClientQuotaAlteration.Op("consumer_byte_rate", 1000.0))); + ClientQuotaAlteration alteration = new ClientQuotaAlteration(entity, singletonList(new ClientQuotaAlteration.Op("consumer_byte_rate", 1000.0))); e = assertThrows(ExecutionException.class, - () -> env.adminClient().alterClientQuotas(asList(alteration)).all().get()); + () -> env.adminClient().alterClientQuotas(singletonList(alteration)).all().get()); assertInstanceOf(AuthenticationException.class, e.getCause(), "Expected an authentication error, but got " + Utils.stackTrace(e)); @@ -1818,9 +1811,9 @@ public void testDeleteAcls() throws Exception { .setThrottleTimeMs(0) .setFilterResults(asList( new DeleteAclsResponseData.DeleteAclsFilterResult() - .setMatchingAcls(asList(DeleteAclsResponse.matchingAcl(ACL1, ApiError.NONE))), + .setMatchingAcls(singletonList(DeleteAclsResponse.matchingAcl(ACL1, ApiError.NONE))), new DeleteAclsResponseData.DeleteAclsFilterResult() - .setMatchingAcls(asList(DeleteAclsResponse.matchingAcl(ACL2, ApiError.NONE))))), + .setMatchingAcls(singletonList(DeleteAclsResponse.matchingAcl(ACL2, ApiError.NONE))))), ApiKeys.DELETE_ACLS.latestVersion())); results = env.adminClient().deleteAcls(asList(FILTER1, FILTER2)); Collection deleted = results.all().get(); @@ -1894,11 +1887,11 @@ public void testDescribeBrokerConfigs() throws Exception { try (AdminClientUnitTestEnv env = mockClientEnv()) { env.kafkaClient().setNodeApiVersions(NodeApiVersions.create()); env.kafkaClient().prepareResponseFrom(new DescribeConfigsResponse( - new DescribeConfigsResponseData().setResults(asList(new DescribeConfigsResponseData.DescribeConfigsResult() + new DescribeConfigsResponseData().setResults(singletonList(new DescribeConfigsResponseData.DescribeConfigsResult() .setResourceName(broker0Resource.name()).setResourceType(broker0Resource.type().id()).setErrorCode(Errors.NONE.code()) .setConfigs(emptyList())))), env.cluster().nodeById(0)); env.kafkaClient().prepareResponseFrom(new DescribeConfigsResponse( - new DescribeConfigsResponseData().setResults(asList(new DescribeConfigsResponseData.DescribeConfigsResult() + new DescribeConfigsResponseData().setResults(singletonList(new DescribeConfigsResponseData.DescribeConfigsResult() .setResourceName(broker1Resource.name()).setResourceType(broker1Resource.type().id()).setErrorCode(Errors.NONE.code()) .setConfigs(emptyList())))), env.cluster().nodeById(1)); Map> result = env.adminClient().describeConfigs(asList( @@ -1939,9 +1932,9 @@ public void testDescribeConfigsPartialResponse() { try (AdminClientUnitTestEnv env = mockClientEnv()) { env.kafkaClient().setNodeApiVersions(NodeApiVersions.create()); env.kafkaClient().prepareResponse(new DescribeConfigsResponse( - new DescribeConfigsResponseData().setResults(asList(new DescribeConfigsResponseData.DescribeConfigsResult() - .setResourceName(topic.name()).setResourceType(topic.type().id()).setErrorCode(Errors.NONE.code()) - .setConfigs(emptyList()))))); + new DescribeConfigsResponseData().setResults(singletonList(new DescribeConfigsResponseData.DescribeConfigsResult() + .setResourceName(topic.name()).setResourceType(topic.type().id()).setErrorCode(Errors.NONE.code()) + .setConfigs(emptyList()))))); Map> result = env.adminClient().describeConfigs(asList( topic, topic2)).values(); @@ -1964,9 +1957,9 @@ public void testDescribeConfigsUnrequested() throws Exception { new DescribeConfigsResponseData.DescribeConfigsResult() .setResourceName(unrequested.name()).setResourceType(unrequested.type().id()).setErrorCode(Errors.NONE.code()) .setConfigs(emptyList()))))); - Map> result = env.adminClient().describeConfigs(asList( + Map> result = env.adminClient().describeConfigs(singletonList( topic)).values(); - assertEquals(new HashSet<>(asList(topic)), result.keySet()); + assertEquals(new HashSet<>(singletonList(topic)), result.keySet()); assertNotNull(result.get(topic).get()); assertNull(result.get(unrequested)); } @@ -2331,10 +2324,10 @@ public void testDescribeReplicaLogDirsUnexpected() throws ExecutionException, In prepareDescribeLogDirsResult(unexpected, broker1log1, broker1Log1PartitionSize, broker1Log1OffsetLag, true)))), env.cluster().nodeById(expected.brokerId())); - DescribeReplicaLogDirsResult result = env.adminClient().describeReplicaLogDirs(asList(expected)); + DescribeReplicaLogDirsResult result = env.adminClient().describeReplicaLogDirs(singletonList(expected)); Map> values = result.values(); - assertEquals(TestUtils.toSet(asList(expected)), values.keySet()); + assertEquals(TestUtils.toSet(singletonList(expected)), values.keySet()); assertNotNull(values.get(expected)); assertEquals(broker1log0, values.get(expected).get().getCurrentReplicaLogDir()); @@ -2360,7 +2353,7 @@ public void testCreatePartitions() throws Exception { Map counts = new HashMap<>(); counts.put("my_topic", NewPartitions.increaseTo(3)); - counts.put("other_topic", NewPartitions.increaseTo(3, asList(asList(2), asList(3)))); + counts.put("other_topic", NewPartitions.increaseTo(3, asList(singletonList(2), singletonList(3)))); CreatePartitionsResult results = env.adminClient().createPartitions(counts); Map> values = results.values(); @@ -3006,12 +2999,12 @@ public void testListConsumerGroupsWithTypes() throws Exception { expectListGroupsRequestWithFilters(singleton(ConsumerGroupState.STABLE.toString()), Collections.emptySet()), new ListGroupsResponse(new ListGroupsResponseData() .setErrorCode(Errors.NONE.code()) - .setGroups(Arrays.asList( - new ListGroupsResponseData.ListedGroup() - .setGroupId("group-1") - .setProtocolType(ConsumerProtocol.PROTOCOL_TYPE) - .setGroupState("Stable") - .setGroupType(GroupType.CLASSIC.toString())))), + .setGroups(singletonList( + new ListGroupsResponseData.ListedGroup() + .setGroupId("group-1") + .setProtocolType(ConsumerProtocol.PROTOCOL_TYPE) + .setGroupState("Stable") + .setGroupType(GroupType.CLASSIC.toString())))), env.cluster().nodeById(0)); final ListConsumerGroupsOptions options = new ListConsumerGroupsOptions().inStates(singleton(ConsumerGroupState.STABLE)); @@ -3567,13 +3560,13 @@ public void testDescribeNonConsumerGroups() throws Exception { DescribeGroupsResponseData data = new DescribeGroupsResponseData(); data.groups().add(DescribeGroupsResponse.groupMetadata( - GROUP_ID, - Errors.NONE, - "", - "non-consumer", - "", - asList(), - Collections.emptySet())); + GROUP_ID, + Errors.NONE, + "", + "non-consumer", + "", + emptyList(), + Collections.emptySet())); env.kafkaClient().prepareResponse(new DescribeGroupsResponse(data)); @@ -5379,12 +5372,12 @@ public void testListOffsets() throws Exception { pInfos.add(new PartitionInfo("qux", 0, node0, new Node[]{node0}, new Node[]{node0})); final Cluster cluster = new Cluster( - "mockClusterId", - Arrays.asList(node0), - pInfos, - Collections.emptySet(), - Collections.emptySet(), - node0); + "mockClusterId", + singletonList(node0), + pInfos, + Collections.emptySet(), + Collections.emptySet(), + node0); final TopicPartition tp0 = new TopicPartition("foo", 0); final TopicPartition tp1 = new TopicPartition("bar", 0); @@ -5474,7 +5467,7 @@ public void testListOffsetsRetriableErrors() throws Exception { ListOffsetsTopicResponse t2 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp2, Errors.NONE, -1L, 456L, 654); responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t2)); + .setTopics(singletonList(t2)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node1); // metadata refresh because of LEADER_NOT_AVAILABLE @@ -5483,7 +5476,7 @@ public void testListOffsetsRetriableErrors() throws Exception { t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NONE, -1L, 345L, 543); responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node0); Map partitions = new HashMap<>(); @@ -5533,7 +5526,7 @@ public void testListOffsetsNonRetriableErrors() throws Exception { ListOffsetsTopicResponse t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.TOPIC_AUTHORIZATION_FAILED, -1L, -1L, -1); ListOffsetsResponseData responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponse(new ListOffsetsResponse(responseData)); Map partitions = new HashMap<>(); @@ -5603,7 +5596,7 @@ public void testListOffsetsMaxTimestampUnsupportedMultipleOffsetSpec() throws Ex ListOffsetsTopicResponse topicResponse = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp1, Errors.NONE, -1L, 345L, 543); ListOffsetsResponseData responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(topicResponse)); + .setTopics(singletonList(topicResponse)); env.kafkaClient().prepareResponseFrom( // ensure that no max timestamp requests are retried request -> request instanceof ListOffsetsRequest && ((ListOffsetsRequest) request).topics().stream() @@ -5771,7 +5764,7 @@ public void testListOffsetsNonMaxTimestampDowngradedImmediately() throws Excepti ListOffsetsTopicResponse t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NONE, -1L, 123L, 321); ListOffsetsResponseData responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); // listoffsets response from broker 0 env.kafkaClient().prepareResponse( @@ -6096,13 +6089,13 @@ public void testListOffsetsMetadataRetriableErrors() throws Exception { ListOffsetsTopicResponse t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NONE, -1L, 345L, 543); ListOffsetsResponseData responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node0); // listoffsets response from broker 1 ListOffsetsTopicResponse t1 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp1, Errors.NONE, -1L, 789L, 987); responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t1)); + .setTopics(singletonList(t1)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node1); Map partitions = new HashMap<>(); @@ -6165,13 +6158,13 @@ public void testListOffsetsWithMultiplePartitionsLeaderChange() throws Exception t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NONE, -1L, 345L, 543); responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node1); t1 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp1, Errors.NONE, -2L, 123L, 456); responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t1)); + .setTopics(singletonList(t1)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node2); Map partitions = new HashMap<>(); @@ -6211,7 +6204,7 @@ public void testListOffsetsWithLeaderChange() throws Exception { ListOffsetsTopicResponse t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NOT_LEADER_OR_FOLLOWER, -1L, 345L, 543); ListOffsetsResponseData responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node0); // updating leader from node0 to node1 and metadata refresh because of NOT_LEADER_OR_FOLLOWER @@ -6225,7 +6218,7 @@ public void testListOffsetsWithLeaderChange() throws Exception { t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NONE, -2L, 123L, 456); responseData = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(responseData), node1); Map partitions = new HashMap<>(); @@ -6335,7 +6328,7 @@ public void testListOffsetsPartialResponse() throws Exception { ListOffsetsTopicResponse t0 = ListOffsetsResponse.singletonListOffsetsTopicResponse(tp0, Errors.NONE, -2L, 123L, 456); ListOffsetsResponseData data = new ListOffsetsResponseData() .setThrottleTimeMs(0) - .setTopics(Arrays.asList(t0)); + .setTopics(singletonList(t0)); env.kafkaClient().prepareResponseFrom(new ListOffsetsResponse(data), node0); Map partitions = new HashMap<>(); @@ -6370,7 +6363,7 @@ public void testSuccessfulRetryAfterRequestTimeout() throws Exception { Node node0 = new Node(0, "localhost", 8121); nodes.put(0, node0); Cluster cluster = new Cluster("mockClusterId", nodes.values(), - Arrays.asList(new PartitionInfo("foo", 0, node0, new Node[]{node0}, new Node[]{node0})), + singletonList(new PartitionInfo("foo", 0, node0, new Node[]{node0}, new Node[]{node0})), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), nodes.get(0)); @@ -6425,7 +6418,7 @@ private void testApiTimeout(int requestTimeoutMs, Node node0 = new Node(0, "localhost", 8121); nodes.put(0, node0); Cluster cluster = new Cluster("mockClusterId", nodes.values(), - Arrays.asList(new PartitionInfo("foo", 0, node0, new Node[]{node0}, new Node[]{node0})), + singletonList(new PartitionInfo("foo", 0, node0, new Node[]{node0}, new Node[]{node0})), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), nodes.get(0)); @@ -6474,7 +6467,7 @@ public void testRequestTimeoutExceedingDefaultApiTimeout() throws Exception { Node node0 = new Node(0, "localhost", 8121); nodes.put(0, node0); Cluster cluster = new Cluster("mockClusterId", nodes.values(), - Arrays.asList(new PartitionInfo("foo", 0, node0, new Node[]{node0}, new Node[]{node0})), + singletonList(new PartitionInfo("foo", 0, node0, new Node[]{node0}, new Node[]{node0})), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), nodes.get(0)); @@ -6532,7 +6525,7 @@ public void testDescribeClientQuotas() throws Exception { env.kafkaClient().prepareResponse(DescribeClientQuotasResponse.fromQuotaEntities(responseData, 0)); - ClientQuotaFilter filter = ClientQuotaFilter.contains(asList(ClientQuotaFilterComponent.ofEntity(ClientQuotaEntity.USER, value))); + ClientQuotaFilter filter = ClientQuotaFilter.contains(singletonList(ClientQuotaFilterComponent.ofEntity(ClientQuotaEntity.USER, value))); DescribeClientQuotasResult result = env.adminClient().describeClientQuotas(filter); Map> resultData = result.entities().get(); @@ -6790,7 +6783,7 @@ public void testAlterUserScramCredentialsUnknownMechanism() { ScramMechanism user2ScramMechanism0 = ScramMechanism.SCRAM_SHA_256; AlterUserScramCredentialsResponseData responseData = new AlterUserScramCredentialsResponseData(); - responseData.setResults(Arrays.asList( + responseData.setResults(singletonList( new AlterUserScramCredentialsResponseData.AlterUserScramCredentialsResult().setUser(user2Name))); env.kafkaClient().prepareResponse(new AlterUserScramCredentialsResponse(responseData)); @@ -7167,7 +7160,7 @@ public void testRetryDescribeTransactionsAfterNotCoordinatorError() throws Excep env.kafkaClient().prepareResponse( request -> request instanceof FindCoordinatorRequest, new FindCoordinatorResponse(new FindCoordinatorResponseData() - .setCoordinators(Arrays.asList(new FindCoordinatorResponseData.Coordinator() + .setCoordinators(singletonList(new FindCoordinatorResponseData.Coordinator() .setKey(transactionalId) .setErrorCode(Errors.NONE.code()) .setNodeId(coordinator1.id()) @@ -7197,7 +7190,7 @@ public void testRetryDescribeTransactionsAfterNotCoordinatorError() throws Excep env.kafkaClient().prepareResponse( request -> request instanceof FindCoordinatorRequest, new FindCoordinatorResponse(new FindCoordinatorResponseData() - .setCoordinators(Arrays.asList(new FindCoordinatorResponseData.Coordinator() + .setCoordinators(singletonList(new FindCoordinatorResponseData.Coordinator() .setKey(transactionalId) .setErrorCode(Errors.NONE.code()) .setNodeId(coordinator2.id()) diff --git a/clients/src/test/java/org/apache/kafka/clients/admin/MockAdminClient.java b/clients/src/test/java/org/apache/kafka/clients/admin/MockAdminClient.java index 6cbc86cb0dd96..f72362715e26e 100644 --- a/clients/src/test/java/org/apache/kafka/clients/admin/MockAdminClient.java +++ b/clients/src/test/java/org/apache/kafka/clients/admin/MockAdminClient.java @@ -105,7 +105,7 @@ public class MockAdminClient extends AdminClient { private KafkaException listConsumerGroupOffsetsException; - private Map mockMetrics = new HashMap<>(); + private final Map mockMetrics = new HashMap<>(); private final List allTokens = new ArrayList<>(); diff --git a/clients/src/test/java/org/apache/kafka/clients/admin/internals/CoordinatorStrategyTest.java b/clients/src/test/java/org/apache/kafka/clients/admin/internals/CoordinatorStrategyTest.java index fc52e9e6717ed..823d4b39b1e9a 100644 --- a/clients/src/test/java/org/apache/kafka/clients/admin/internals/CoordinatorStrategyTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/admin/internals/CoordinatorStrategyTest.java @@ -90,7 +90,7 @@ public void testBuildOldLookupRequestRequiresAtLeastOneKey() { strategy.disableBatch(); assertThrows(IllegalArgumentException.class, () -> strategy.buildRequest( - new HashSet<>(Arrays.asList(CoordinatorKey.byTransactionalId("txnid"))))); + new HashSet<>(Collections.singletonList(CoordinatorKey.byTransactionalId("txnid"))))); } @Test diff --git a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java index b1db0297a120b..043cc211da5b4 100644 --- a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java @@ -52,7 +52,6 @@ import org.mockito.Mockito; import java.util.ArrayList; -import java.util.Arrays; import java.util.Collections; import java.util.HashMap; import java.util.HashSet; @@ -1336,13 +1335,13 @@ public ClientResponse mockOffsetCommitResponse(String topic, long receivedTimeMs, Errors error) { OffsetCommitResponseData responseData = new OffsetCommitResponseData() - .setTopics(Arrays.asList( - new OffsetCommitResponseData.OffsetCommitResponseTopic() - .setName(topic) - .setPartitions(Collections.singletonList( - new OffsetCommitResponseData.OffsetCommitResponsePartition() - .setErrorCode(error.code()) - .setPartitionIndex(partition))))); + .setTopics(Collections.singletonList( + new OffsetCommitResponseData.OffsetCommitResponseTopic() + .setName(topic) + .setPartitions(Collections.singletonList( + new OffsetCommitResponseData.OffsetCommitResponsePartition() + .setErrorCode(error.code()) + .setPartitionIndex(partition))))); OffsetCommitResponse response = mock(OffsetCommitResponse.class); when(response.data()).thenReturn(responseData); return new ClientResponse( @@ -1362,13 +1361,13 @@ public ClientResponse mockOffsetCommitResponseDisconnected(String topic, int par short apiKeyVersion, NetworkClientDelegate.UnsentRequest unsentRequest) { OffsetCommitResponseData responseData = new OffsetCommitResponseData() - .setTopics(Arrays.asList( - new OffsetCommitResponseData.OffsetCommitResponseTopic() - .setName(topic) - .setPartitions(Collections.singletonList( - new OffsetCommitResponseData.OffsetCommitResponsePartition() - .setErrorCode(Errors.NONE.code()) - .setPartitionIndex(partition))))); + .setTopics(Collections.singletonList( + new OffsetCommitResponseData.OffsetCommitResponseTopic() + .setName(topic) + .setPartitions(Collections.singletonList( + new OffsetCommitResponseData.OffsetCommitResponsePartition() + .setErrorCode(Errors.NONE.code()) + .setPartitionIndex(partition))))); OffsetCommitResponse response = mock(OffsetCommitResponse.class); when(response.data()).thenReturn(responseData); return new ClientResponse( diff --git a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java index 954ed1c11e09b..541c6a2a04adc 100644 --- a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java @@ -162,13 +162,13 @@ public abstract class ConsumerCoordinatorTest { private final String consumerId2 = "consumer2"; private MockClient client; - private MetadataResponse metadataResponse = RequestTestUtils.metadataUpdateWith(1, new HashMap() { + private final MetadataResponse metadataResponse = RequestTestUtils.metadataUpdateWith(1, new HashMap() { { put(topic1, 1); put(topic2, 1); } }); - private Node node = metadataResponse.brokers().iterator().next(); + private final Node node = metadataResponse.brokers().iterator().next(); private SubscriptionState subscriptions; private ConsumerMetadata metadata; private Metrics metrics; @@ -311,7 +311,7 @@ public void testPerformAssignmentShouldUpdateGroupSubscriptionAfterAssignmentIfN List> capturedTopics = topicsCaptor.getAllValues(); // expected the final group subscribed topics to be updated to "topic1" - Set expectedTopicsGotCalled = new HashSet<>(Arrays.asList(topic1)); + Set expectedTopicsGotCalled = new HashSet<>(singletonList(topic1)); assertEquals(expectedTopicsGotCalled, capturedTopics.get(0)); } @@ -389,8 +389,8 @@ public void testPerformAssignmentShouldValidateCooperativeAssignment() { // simulate the custom cooperative assignor didn't revoke the partition first before assign to other consumer Map> assignment = new HashMap<>(); - assignment.put(consumerId, Arrays.asList(t1p)); - assignment.put(consumerId2, Arrays.asList(t2p)); + assignment.put(consumerId, singletonList(t1p)); + assignment.put(consumerId2, singletonList(t2p)); partitionAssignor.prepare(assignment); try (ConsumerCoordinator coordinator = buildCoordinator(rebalanceConfig, new Metrics(), assignors, false, mockSubscriptionState)) { @@ -450,8 +450,8 @@ public String name() { // simulate the cooperative sticky assignor do the assignment with out-of-date ownedPartition Map> assignment = new HashMap<>(); - assignment.put(consumerId, Arrays.asList(t1p)); - assignment.put(consumerId2, Arrays.asList(t2p)); + assignment.put(consumerId, singletonList(t1p)); + assignment.put(consumerId2, singletonList(t2p)); mockCooperativeStickyAssignor.prepare(assignment); try (ConsumerCoordinator coordinator = buildCoordinator(rebalanceConfig, new Metrics(), assignorsWithCooperativeStickyAssignor, false, mockSubscriptionState)) { @@ -979,7 +979,7 @@ public void testNormalJoinGroupLeader() { final String consumerId = "leader"; final Set subscription = singleton(topic1); final List owned = Collections.emptyList(); - final List assigned = Arrays.asList(t1p); + final List assigned = singletonList(t1p); subscriptions.subscribe(singleton(topic1), Optional.of(rebalanceListener)); @@ -1016,9 +1016,9 @@ public void testOutdatedCoordinatorAssignment() { final String consumerId = "outdated_assignment"; final List owned = Collections.emptyList(); final List oldSubscription = singletonList(topic2); - final List oldAssignment = Arrays.asList(t2p); + final List oldAssignment = singletonList(t2p); final List newSubscription = singletonList(topic1); - final List newAssignment = Arrays.asList(t1p); + final List newAssignment = singletonList(t1p); subscriptions.subscribe(toSet(oldSubscription), Optional.of(rebalanceListener)); @@ -2051,7 +2051,7 @@ public void testUpdateMetadataDuringRebalance() { // prepare initial rebalance Map> memberSubscriptions = singletonMap(consumerId, topics); - partitionAssignor.prepare(singletonMap(consumerId, Arrays.asList(tp1))); + partitionAssignor.prepare(singletonMap(consumerId, singletonList(tp1))); client.prepareResponse(joinGroupLeaderResponse(1, consumerId, memberSubscriptions, Errors.NONE)); client.prepareResponse(body -> { @@ -2254,7 +2254,7 @@ private void testInternalTopicInclusion(boolean includeInternalTopics) { public void testRejoinGroup() { String otherTopic = "otherTopic"; final List owned = Collections.emptyList(); - final List assigned = Arrays.asList(t1p); + final List assigned = singletonList(t1p); subscriptions.subscribe(singleton(topic1), Optional.of(rebalanceListener)); @@ -2286,7 +2286,7 @@ public void testRejoinGroup() { public void testDisconnectInJoin() { subscriptions.subscribe(singleton(topic1), Optional.of(rebalanceListener)); final List owned = Collections.emptyList(); - final List assigned = Arrays.asList(t1p); + final List assigned = singletonList(t1p); client.prepareResponse(groupCoordinatorResponse(node, Errors.NONE)); coordinator.ensureCoordinatorReady(time.timer(Long.MAX_VALUE)); diff --git a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerInterceptorsTest.java b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerInterceptorsTest.java index 4331e72054177..27404877ec014 100644 --- a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerInterceptorsTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerInterceptorsTest.java @@ -54,7 +54,7 @@ public class ConsumerInterceptorsTest { * Test consumer interceptor that filters records in onConsume() intercept */ private class FilterConsumerInterceptor implements ConsumerInterceptor { - private int filterPartition; + private final int filterPartition; private boolean throwExceptionOnConsume = false; private boolean throwExceptionOnCommit = false; diff --git a/clients/src/test/java/org/apache/kafka/clients/producer/KafkaProducerTest.java b/clients/src/test/java/org/apache/kafka/clients/producer/KafkaProducerTest.java index 7d4aa5e3a85d6..643dffb617918 100644 --- a/clients/src/test/java/org/apache/kafka/clients/producer/KafkaProducerTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/producer/KafkaProducerTest.java @@ -2070,7 +2070,7 @@ public void testCallbackAndInterceptorHandleError() { String invalidTopicName = "topic abc"; // Invalid topic name due to space ProducerInterceptors producerInterceptors = - new ProducerInterceptors<>(Arrays.asList(new MockProducerInterceptor())); + new ProducerInterceptors<>(Collections.singletonList(new MockProducerInterceptor())); try (Producer producer = kafkaProducer(configs, new StringSerializer(), new StringSerializer(), producerMetadata, client, producerInterceptors, time)) { diff --git a/clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java b/clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java index 4719bf99c1c0a..ff4e2eec498da 100644 --- a/clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/producer/internals/RecordAccumulatorTest.java @@ -1486,7 +1486,7 @@ public void testReadyAndDrainWhenABatchIsBeingRetried() throws InterruptedExcept int part1LeaderEpoch = 100; // Create cluster metadata, partition1 being hosted by node1 PartitionMetadata part1Metadata = new PartitionMetadata(Errors.NONE, tp1, Optional.of(node1.id()), Optional.of(part1LeaderEpoch), null, null, null); - MetadataSnapshot metadataCache = new MetadataSnapshot(null, nodes, Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + MetadataSnapshot metadataCache = new MetadataSnapshot(null, nodes, Collections.singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); int batchSize = 10; int lingerMs = 10; @@ -1528,7 +1528,7 @@ deliveryTimeoutMs, metrics, metricGrpName, time, new ApiVersions(), null, // Try to drain from node1, it should return no batches. Map> batches = accum.drain(metadataCache, - new HashSet<>(Arrays.asList(node1)), 999999 /* maxSize */, now); + new HashSet<>(Collections.singletonList(node1)), 999999 /* maxSize */, now); assertTrue(batches.containsKey(node1.id()) && batches.get(node1.id()).isEmpty(), "No batches ready to be drained on Node1"); } @@ -1539,7 +1539,7 @@ deliveryTimeoutMs, metrics, metricGrpName, time, new ApiVersions(), null, part1LeaderEpoch++; // Create cluster metadata, with new leader epoch. part1Metadata = new PartitionMetadata(Errors.NONE, tp1, Optional.of(node1.id()), Optional.of(part1LeaderEpoch), null, null, null); - metadataCache = new MetadataSnapshot(null, nodes, Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + metadataCache = new MetadataSnapshot(null, nodes, Collections.singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); RecordAccumulator.ReadyCheckResult result = accum.ready(metadataCache, now); assertTrue(result.readyNodes.contains(node1), "Node1 is ready"); @@ -1559,7 +1559,7 @@ deliveryTimeoutMs, metrics, metricGrpName, time, new ApiVersions(), null, now += 2 * retryBackoffMaxMs; // Create cluster metadata, with new leader epoch. part1Metadata = new PartitionMetadata(Errors.NONE, tp1, Optional.of(node1.id()), Optional.of(part1LeaderEpoch), null, null, null); - metadataCache = new MetadataSnapshot(null, nodes, Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + metadataCache = new MetadataSnapshot(null, nodes, Collections.singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); RecordAccumulator.ReadyCheckResult result = accum.ready(metadataCache, now); assertTrue(result.readyNodes.contains(node1), "Node1 is ready"); @@ -1580,7 +1580,7 @@ deliveryTimeoutMs, metrics, metricGrpName, time, new ApiVersions(), null, part1LeaderEpoch++; // Create cluster metadata, with new leader epoch. part1Metadata = new PartitionMetadata(Errors.NONE, tp1, Optional.of(node1.id()), Optional.of(part1LeaderEpoch), null, null, null); - metadataCache = new MetadataSnapshot(null, nodes, Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + metadataCache = new MetadataSnapshot(null, nodes, Collections.singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); RecordAccumulator.ReadyCheckResult result = accum.ready(metadataCache, now); assertTrue(result.readyNodes.contains(node1), "Node1 is ready"); @@ -1605,11 +1605,11 @@ public void testDrainWithANodeThatDoesntHostAnyPartitions() { // Create cluster metadata, node2 doesn't host any partitions. PartitionMetadata part1Metadata = new PartitionMetadata(Errors.NONE, tp1, Optional.of(node1.id()), Optional.empty(), null, null, null); - MetadataSnapshot metadataCache = new MetadataSnapshot(null, nodes, Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + MetadataSnapshot metadataCache = new MetadataSnapshot(null, nodes, Collections.singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); // Drain for node2, it should return 0 batches, Map> batches = accum.drain(metadataCache, - new HashSet<>(Arrays.asList(node2)), 999999 /* maxSize */, time.milliseconds()); + new HashSet<>(Collections.singletonList(node2)), 999999 /* maxSize */, time.milliseconds()); assertTrue(batches.get(node2.id()).isEmpty()); } diff --git a/clients/src/test/java/org/apache/kafka/clients/producer/internals/TransactionManagerTest.java b/clients/src/test/java/org/apache/kafka/clients/producer/internals/TransactionManagerTest.java index cdbd7c3f7b919..f112e424a2de7 100644 --- a/clients/src/test/java/org/apache/kafka/clients/producer/internals/TransactionManagerTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/producer/internals/TransactionManagerTest.java @@ -2512,7 +2512,7 @@ public void testAllowDrainInAbortableErrorState() throws InterruptedException { // Try to drain a message destined for tp1, it should get drained. Node node1 = new Node(1, "localhost", 1112); PartitionMetadata part1Metadata = new PartitionMetadata(Errors.NONE, tp1, Optional.of(node1.id()), Optional.empty(), null, null, null); - MetadataSnapshot metadataCache = new MetadataSnapshot(null, Collections.singletonMap(node1.id(), node1), Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + MetadataSnapshot metadataCache = new MetadataSnapshot(null, Collections.singletonMap(node1.id(), node1), singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); appendToAccumulator(tp1); Map> drainedBatches = accumulator.drain(metadataCache, Collections.singleton(node1), Integer.MAX_VALUE, @@ -2533,7 +2533,7 @@ public void testRaiseErrorWhenNoPartitionsPendingOnDrain() throws InterruptedExc appendToAccumulator(tp0); Node node1 = new Node(0, "localhost", 1111); PartitionMetadata part1Metadata = new PartitionMetadata(Errors.NONE, tp0, Optional.of(node1.id()), Optional.empty(), null, null, null); - MetadataSnapshot metadataCache = new MetadataSnapshot(null, Collections.singletonMap(node1.id(), node1), Arrays.asList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); + MetadataSnapshot metadataCache = new MetadataSnapshot(null, Collections.singletonMap(node1.id(), node1), singletonList(part1Metadata), Collections.emptySet(), Collections.emptySet(), Collections.emptySet(), null, Collections.emptyMap()); Set nodes = new HashSet<>(); nodes.add(node1); diff --git a/clients/src/test/java/org/apache/kafka/common/config/ConfigDefTest.java b/clients/src/test/java/org/apache/kafka/common/config/ConfigDefTest.java index b85dd3556e007..8add9c95f3071 100644 --- a/clients/src/test/java/org/apache/kafka/common/config/ConfigDefTest.java +++ b/clients/src/test/java/org/apache/kafka/common/config/ConfigDefTest.java @@ -121,7 +121,7 @@ public void testDefinedTwice() { @Test public void testBadInputs() { testBadInputs(Type.INT, "hello", "42.5", 42.5, Long.MAX_VALUE, Long.toString(Long.MAX_VALUE), new Object()); - testBadInputs(Type.LONG, "hello", "42.5", Long.toString(Long.MAX_VALUE) + "00", new Object()); + testBadInputs(Type.LONG, "hello", "42.5", Long.MAX_VALUE + "00", new Object()); testBadInputs(Type.DOUBLE, "hello", new Object()); testBadInputs(Type.STRING, new Object()); testBadInputs(Type.LIST, 53, new Object()); @@ -242,7 +242,7 @@ public void testParseForValidate() { String errorMessageC = "Missing required configuration \"c\" which has no default value."; ConfigValue configA = new ConfigValue("a", 1, Collections.emptyList(), Collections.emptyList()); ConfigValue configB = new ConfigValue("b", null, Collections.emptyList(), Arrays.asList(errorMessageB, errorMessageB)); - ConfigValue configC = new ConfigValue("c", null, Collections.emptyList(), Arrays.asList(errorMessageC)); + ConfigValue configC = new ConfigValue("c", null, Collections.emptyList(), singletonList(errorMessageC)); ConfigValue configD = new ConfigValue("d", 10, Collections.emptyList(), Collections.emptyList()); expected.put("a", configA); expected.put("b", configB); @@ -253,7 +253,7 @@ public void testParseForValidate() { .define("a", Type.INT, Importance.HIGH, "docs", "group", 1, Width.SHORT, "a", Arrays.asList("b", "c"), new IntegerRecommender(false)) .define("b", Type.INT, Importance.HIGH, "docs", "group", 2, Width.SHORT, "b", new IntegerRecommender(true)) .define("c", Type.INT, Importance.HIGH, "docs", "group", 3, Width.SHORT, "c", new IntegerRecommender(true)) - .define("d", Type.INT, Importance.HIGH, "docs", "group", 4, Width.SHORT, "d", Arrays.asList("b"), new IntegerRecommender(false)); + .define("d", Type.INT, Importance.HIGH, "docs", "group", 4, Width.SHORT, "d", singletonList("b"), new IntegerRecommender(false)); Map props = new HashMap<>(); props.put("a", "1"); @@ -279,7 +279,7 @@ public void testValidate() { ConfigValue configA = new ConfigValue("a", 1, Arrays.asList(1, 2, 3), Collections.emptyList()); ConfigValue configB = new ConfigValue("b", null, Arrays.asList(4, 5), Arrays.asList(errorMessageB, errorMessageB)); - ConfigValue configC = new ConfigValue("c", null, Arrays.asList(4, 5), Arrays.asList(errorMessageC)); + ConfigValue configC = new ConfigValue("c", null, Arrays.asList(4, 5), singletonList(errorMessageC)); ConfigValue configD = new ConfigValue("d", 10, Arrays.asList(1, 2, 3), Collections.emptyList()); expected.put("a", configA); @@ -291,7 +291,7 @@ public void testValidate() { .define("a", Type.INT, Importance.HIGH, "docs", "group", 1, Width.SHORT, "a", Arrays.asList("b", "c"), new IntegerRecommender(false)) .define("b", Type.INT, Importance.HIGH, "docs", "group", 2, Width.SHORT, "b", new IntegerRecommender(true)) .define("c", Type.INT, Importance.HIGH, "docs", "group", 3, Width.SHORT, "c", new IntegerRecommender(true)) - .define("d", Type.INT, Importance.HIGH, "docs", "group", 4, Width.SHORT, "d", Arrays.asList("b"), new IntegerRecommender(false)); + .define("d", Type.INT, Importance.HIGH, "docs", "group", 4, Width.SHORT, "d", singletonList("b"), new IntegerRecommender(false)); Map props = new HashMap<>(); props.put("a", "1"); @@ -313,9 +313,9 @@ public void testValidateMissingConfigKey() { String errorMessageD = "d is referred in the dependents, but not defined."; ConfigValue configA = new ConfigValue("a", 1, Arrays.asList(1, 2, 3), Collections.emptyList()); - ConfigValue configB = new ConfigValue("b", null, Arrays.asList(4, 5), Arrays.asList(errorMessageB)); - ConfigValue configC = new ConfigValue("c", null, Arrays.asList(4, 5), Arrays.asList(errorMessageC)); - ConfigValue configD = new ConfigValue("d", null, Collections.emptyList(), Arrays.asList(errorMessageD)); + ConfigValue configB = new ConfigValue("b", null, Arrays.asList(4, 5), singletonList(errorMessageB)); + ConfigValue configC = new ConfigValue("c", null, Arrays.asList(4, 5), singletonList(errorMessageC)); + ConfigValue configD = new ConfigValue("d", null, Collections.emptyList(), singletonList(errorMessageD)); configD.visible(false); expected.put("a", configA); @@ -343,7 +343,7 @@ public void testValidateMissingConfigKey() { public void testValidateCannotParse() { Map expected = new HashMap<>(); String errorMessageB = "Invalid value non_integer for configuration a: Not a number of type INT"; - ConfigValue configA = new ConfigValue("a", null, Collections.emptyList(), Arrays.asList(errorMessageB)); + ConfigValue configA = new ConfigValue("a", null, Collections.emptyList(), singletonList(errorMessageB)); expected.put("a", configA); ConfigDef def = new ConfigDef().define("a", Type.INT, Importance.HIGH, "docs"); @@ -438,7 +438,7 @@ public void testBaseConfigDefDependents() { // Creating a ConfigDef based on another should compute the correct number of configs with no parent, even // if the base ConfigDef has already computed its parentless configs final ConfigDef baseConfigDef = new ConfigDef().define("a", Type.STRING, Importance.LOW, "docs"); - assertEquals(new HashSet<>(Arrays.asList("a")), baseConfigDef.getConfigsWithNoParent()); + assertEquals(new HashSet<>(singletonList("a")), baseConfigDef.getConfigsWithNoParent()); final ConfigDef configDef = new ConfigDef(baseConfigDef) .define("parent", Type.STRING, Importance.HIGH, "parent docs", "group", 1, Width.LONG, "Parent", singletonList("child")) diff --git a/clients/src/test/java/org/apache/kafka/common/config/provider/DirectoryConfigProviderTest.java b/clients/src/test/java/org/apache/kafka/common/config/provider/DirectoryConfigProviderTest.java index 59949e6043c3e..b351b0e6e6cae 100644 --- a/clients/src/test/java/org/apache/kafka/common/config/provider/DirectoryConfigProviderTest.java +++ b/clients/src/test/java/org/apache/kafka/common/config/provider/DirectoryConfigProviderTest.java @@ -179,12 +179,12 @@ public void testMultipleAllowedPaths() { provider.configure(configs); ConfigData configData = provider.get(subdir); - assertEquals(toSet(asList(subdirFileName)), configData.data().keySet()); + assertEquals(toSet(Collections.singletonList(subdirFileName)), configData.data().keySet()); assertEquals("SUBDIRFILE", configData.data().get(subdirFileName)); assertNull(configData.ttl()); configData = provider.get(siblingDir); - assertEquals(toSet(asList(siblingDirFileName)), configData.data().keySet()); + assertEquals(toSet(Collections.singletonList(siblingDirFileName)), configData.data().keySet()); assertEquals("SIBLINGDIRFILE", configData.data().get(siblingDirFileName)); assertNull(configData.ttl()); } diff --git a/clients/src/test/java/org/apache/kafka/common/metrics/JmxReporterTest.java b/clients/src/test/java/org/apache/kafka/common/metrics/JmxReporterTest.java index a6b2e7f65d8e9..3f71a45628f26 100644 --- a/clients/src/test/java/org/apache/kafka/common/metrics/JmxReporterTest.java +++ b/clients/src/test/java/org/apache/kafka/common/metrics/JmxReporterTest.java @@ -26,7 +26,7 @@ import javax.management.ObjectName; import java.lang.management.ManagementFactory; import java.util.ArrayList; -import java.util.Arrays; +import java.util.Collections; import java.util.HashMap; import java.util.Map; @@ -164,7 +164,7 @@ public void testJmxPrefix() throws Exception { JmxReporter reporter = new JmxReporter(); MetricsContext metricsContext = new KafkaMetricsContext("kafka.server"); MetricConfig metricConfig = new MetricConfig(); - Metrics metrics = new Metrics(metricConfig, new ArrayList<>(Arrays.asList(reporter)), Time.SYSTEM, metricsContext); + Metrics metrics = new Metrics(metricConfig, new ArrayList<>(Collections.singletonList(reporter)), Time.SYSTEM, metricsContext); MBeanServer server = ManagementFactory.getPlatformMBeanServer(); try { @@ -183,7 +183,7 @@ public void testDeprecatedJmxPrefixWithDefaultMetrics() throws Exception { // for backwards compatibility, ensure prefix does not get overridden by the default empty namespace in metricscontext MetricConfig metricConfig = new MetricConfig(); - Metrics metrics = new Metrics(metricConfig, new ArrayList<>(Arrays.asList(reporter)), Time.SYSTEM); + Metrics metrics = new Metrics(metricConfig, new ArrayList<>(Collections.singletonList(reporter)), Time.SYSTEM); MBeanServer server = ManagementFactory.getPlatformMBeanServer(); try { diff --git a/clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java b/clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java index f514281abf314..0713f0ac30a3f 100644 --- a/clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java +++ b/clients/src/test/java/org/apache/kafka/common/metrics/MetricsTest.java @@ -77,7 +77,7 @@ public class MetricsTest { @BeforeEach public void setup() { - this.metrics = new Metrics(config, Arrays.asList(new JmxReporter()), time, true); + this.metrics = new Metrics(config, singletonList(new JmxReporter()), time, true); } @AfterEach @@ -197,7 +197,7 @@ public void testHierarchicalSensors() { assertEquals(1.0 + c1, p2, EPS); assertEquals(1.0 + c1 + c2, p1, EPS); assertEquals(Arrays.asList(child1, child2), metrics.childrenSensors().get(parent1)); - assertEquals(Arrays.asList(child1), metrics.childrenSensors().get(parent2)); + assertEquals(singletonList(child1), metrics.childrenSensors().get(parent2)); assertNull(metrics.childrenSensors().get(grandchild)); } @@ -719,7 +719,7 @@ public void testMetricInstances() { Map childTagsWithValues = new HashMap<>(); childTagsWithValues.put("child-tag", "child-tag-value"); - try (Metrics inherited = new Metrics(new MetricConfig().tags(parentTagsWithValues), Arrays.asList(new JmxReporter()), time, true)) { + try (Metrics inherited = new Metrics(new MetricConfig().tags(parentTagsWithValues), singletonList(new JmxReporter()), time, true)) { MetricName inheritedMetric = inherited.metricInstance(SampleMetrics.METRIC_WITH_INHERITED_TAGS, childTagsWithValues); Map filledOutTags = inheritedMetric.tags(); @@ -787,7 +787,7 @@ public void testConcurrentReadUpdate() { public void testConcurrentReadUpdateReport() { class LockingReporter implements MetricsReporter { - Map activeMetrics = new HashMap<>(); + final Map activeMetrics = new HashMap<>(); @Override public synchronized void init(List metrics) { } @@ -819,7 +819,7 @@ synchronized void processMetrics() { final LockingReporter reporter = new LockingReporter(); this.metrics.close(); - this.metrics = new Metrics(config, Arrays.asList(reporter), new MockTime(10), true); + this.metrics = new Metrics(config, singletonList(reporter), new MockTime(10), true); final Deque sensors = new ConcurrentLinkedDeque<>(); SensorCreator sensorCreator = new SensorCreator(metrics); diff --git a/clients/src/test/java/org/apache/kafka/common/metrics/SensorTest.java b/clients/src/test/java/org/apache/kafka/common/metrics/SensorTest.java index 9254616528fe7..df3eedd176ad9 100644 --- a/clients/src/test/java/org/apache/kafka/common/metrics/SensorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/metrics/SensorTest.java @@ -29,7 +29,6 @@ import org.junit.jupiter.api.Test; import java.util.ArrayList; -import java.util.Arrays; import java.util.Collections; import java.util.List; import java.util.Map; @@ -128,7 +127,7 @@ public void testShouldRecordForTraceLevelSensor() { public void testExpiredSensor() { MetricConfig config = new MetricConfig(); Time mockTime = new MockTime(); - try (Metrics metrics = new Metrics(config, Arrays.asList(new JmxReporter()), mockTime, true)) { + try (Metrics metrics = new Metrics(config, Collections.singletonList(new JmxReporter()), mockTime, true)) { long inactiveSensorExpirationTimeSeconds = 60L; Sensor sensor = new Sensor(metrics, "sensor", null, config, mockTime, inactiveSensorExpirationTimeSeconds, Sensor.RecordingLevel.INFO); diff --git a/clients/src/test/java/org/apache/kafka/common/metrics/stats/FrequenciesTest.java b/clients/src/test/java/org/apache/kafka/common/metrics/stats/FrequenciesTest.java index 344ade22a66ed..c46691542349e 100644 --- a/clients/src/test/java/org/apache/kafka/common/metrics/stats/FrequenciesTest.java +++ b/clients/src/test/java/org/apache/kafka/common/metrics/stats/FrequenciesTest.java @@ -29,7 +29,6 @@ import org.junit.jupiter.api.BeforeEach; import org.junit.jupiter.api.Test; -import java.util.Arrays; import java.util.Collections; import static org.junit.jupiter.api.Assertions.assertEquals; @@ -46,7 +45,7 @@ public class FrequenciesTest { public void setup() { config = new MetricConfig().eventWindow(50).samples(2); time = new MockTime(); - metrics = new Metrics(config, Arrays.asList(new JmxReporter()), time, true); + metrics = new Metrics(config, Collections.singletonList(new JmxReporter()), time, true); } @AfterEach diff --git a/clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java b/clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java index 25a240c2ede18..a46a38af30c13 100644 --- a/clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/network/SelectorTest.java @@ -955,7 +955,7 @@ public void testWriteCompletesSendWithNoBytesWritten() throws IOException { NetworkSend send = new NetworkSend("destination", new ByteBufferSend(ByteBuffer.allocate(0))); when(channel.maybeCompleteSend()).thenReturn(send); selector.write(channel); - assertEquals(asList(send), selector.completedSends()); + assertEquals(Collections.singletonList(send), selector.completedSends()); } /** diff --git a/clients/src/test/java/org/apache/kafka/common/network/SslTransportLayerTest.java b/clients/src/test/java/org/apache/kafka/common/network/SslTransportLayerTest.java index aeb37af931369..e56ce7abbad0b 100644 --- a/clients/src/test/java/org/apache/kafka/common/network/SslTransportLayerTest.java +++ b/clients/src/test/java/org/apache/kafka/common/network/SslTransportLayerTest.java @@ -55,7 +55,6 @@ import java.nio.channels.SocketChannel; import java.nio.charset.StandardCharsets; import java.util.ArrayList; -import java.util.Arrays; import java.util.Collection; import java.util.Collections; import java.util.HashMap; @@ -105,7 +104,7 @@ private static class Args { private CertStores clientCertStores; private Map sslClientConfigs; private Map sslServerConfigs; - private Map sslConfigOverrides; + private final Map sslConfigOverrides; public Args(String tlsProtocol, boolean useInlinePem) throws Exception { this.tlsProtocol = tlsProtocol; @@ -621,7 +620,7 @@ public void testTlsDefaults(Args args) throws Exception { /** Checks connection failed using the specified {@code tlsVersion}. */ private void checkAuthenticationFailed(Args args, String node, String tlsVersion) throws IOException { - args.sslClientConfigs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, Arrays.asList(tlsVersion)); + args.sslClientConfigs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, Collections.singletonList(tlsVersion)); createSelector(args.sslClientConfigs); InetSocketAddress addr = new InetSocketAddress("localhost", server.port()); selector.connect(node, addr, BUFFER_SIZE, BUFFER_SIZE); @@ -640,10 +639,10 @@ public void testUnsupportedCiphers(Args args) throws Exception { SSLContext context = SSLContext.getInstance(args.tlsProtocol); context.init(null, null, null); String[] cipherSuites = context.getDefaultSSLParameters().getCipherSuites(); - args.sslServerConfigs.put(SslConfigs.SSL_CIPHER_SUITES_CONFIG, Arrays.asList(cipherSuites[0])); + args.sslServerConfigs.put(SslConfigs.SSL_CIPHER_SUITES_CONFIG, Collections.singletonList(cipherSuites[0])); server = createEchoServer(args, SecurityProtocol.SSL); - args.sslClientConfigs.put(SslConfigs.SSL_CIPHER_SUITES_CONFIG, Arrays.asList(cipherSuites[1])); + args.sslClientConfigs.put(SslConfigs.SSL_CIPHER_SUITES_CONFIG, Collections.singletonList(cipherSuites[1])); createSelector(args.sslClientConfigs); checkAuthenticationFailed(args, "1", args.tlsProtocol); diff --git a/clients/src/test/java/org/apache/kafka/common/network/SslTransportTls12Tls13Test.java b/clients/src/test/java/org/apache/kafka/common/network/SslTransportTls12Tls13Test.java index 4f6e4b3aced70..425ab23532bb8 100644 --- a/clients/src/test/java/org/apache/kafka/common/network/SslTransportTls12Tls13Test.java +++ b/clients/src/test/java/org/apache/kafka/common/network/SslTransportTls12Tls13Test.java @@ -139,7 +139,7 @@ public void testCiphersSuiteForTls12() throws Exception { /** Checks connection failed using the specified {@code tlsVersion}. */ private void checkAuthenticationFailed() throws IOException, InterruptedException { - sslClientConfigs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, Arrays.asList("TLSv1.3")); + sslClientConfigs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, Collections.singletonList("TLSv1.3")); createSelector(sslClientConfigs); InetSocketAddress addr = new InetSocketAddress("localhost", server.port()); selector.connect("0", addr, BUFFER_SIZE, BUFFER_SIZE); diff --git a/clients/src/test/java/org/apache/kafka/common/network/Tls12SelectorTest.java b/clients/src/test/java/org/apache/kafka/common/network/Tls12SelectorTest.java index 7169b2ec51706..750f75f50753c 100644 --- a/clients/src/test/java/org/apache/kafka/common/network/Tls12SelectorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/network/Tls12SelectorTest.java @@ -17,7 +17,6 @@ package org.apache.kafka.common.network; -import static java.util.Arrays.asList; import static org.junit.jupiter.api.Assertions.assertTrue; import java.io.File; @@ -25,6 +24,7 @@ import java.net.InetSocketAddress; import java.security.GeneralSecurityException; import java.util.ArrayList; +import java.util.Collections; import java.util.List; import java.util.Map; import org.apache.kafka.common.config.SslConfigs; @@ -38,7 +38,7 @@ protected Map createSslClientConfigs(File trustStoreFile) throws GeneralSecurityException, IOException { Map configs = TestSslUtils.createSslConfig(false, false, Mode.CLIENT, trustStoreFile, "client"); - configs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, asList("TLSv1.2")); + configs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, Collections.singletonList("TLSv1.2")); return configs; } diff --git a/clients/src/test/java/org/apache/kafka/common/network/Tls13SelectorTest.java b/clients/src/test/java/org/apache/kafka/common/network/Tls13SelectorTest.java index db69c2fa8ea1e..2313ec4748e96 100644 --- a/clients/src/test/java/org/apache/kafka/common/network/Tls13SelectorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/network/Tls13SelectorTest.java @@ -25,6 +25,7 @@ import java.net.InetSocketAddress; import java.security.GeneralSecurityException; import java.util.ArrayList; +import java.util.Collections; import java.util.List; import java.util.Map; import java.util.stream.Collectors; @@ -42,7 +43,7 @@ public class Tls13SelectorTest extends SslSelectorTest { protected Map createSslClientConfigs(File trustStoreFile) throws GeneralSecurityException, IOException { Map configs = TestSslUtils.createSslConfig(false, false, Mode.CLIENT, trustStoreFile, "client"); - configs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, asList("TLSv1.3")); + configs.put(SslConfigs.SSL_ENABLED_PROTOCOLS_CONFIG, Collections.singletonList("TLSv1.3")); return configs; } diff --git a/clients/src/test/java/org/apache/kafka/common/protocol/MessageUtilTest.java b/clients/src/test/java/org/apache/kafka/common/protocol/MessageUtilTest.java index 4795798908ff8..b3181907b52a4 100755 --- a/clients/src/test/java/org/apache/kafka/common/protocol/MessageUtilTest.java +++ b/clients/src/test/java/org/apache/kafka/common/protocol/MessageUtilTest.java @@ -46,7 +46,7 @@ public void testDeepToString() { assertEquals("[1, 2, 3]", MessageUtil.deepToString(Arrays.asList(1, 2, 3).iterator())); assertEquals("[foo]", - MessageUtil.deepToString(Arrays.asList("foo").iterator())); + MessageUtil.deepToString(Collections.singletonList("foo").iterator())); } @Test diff --git a/clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsBuilderTest.java b/clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsBuilderTest.java index 49c44459ca9e8..5822f89efabb5 100644 --- a/clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsBuilderTest.java +++ b/clients/src/test/java/org/apache/kafka/common/record/MemoryRecordsBuilderTest.java @@ -759,7 +759,7 @@ public void testBuffersDereferencedOnClose(Args args) { // Ignore memory usage during initialization if (iterations == 2) startMem = memUsed; - else if (iterations > 2 && memUsed < (iterations - 2) * 1024) + else if (iterations > 2 && memUsed < (iterations - 2) * 1024L) break; } assertTrue(iterations < 100, "Memory usage too high: " + memUsed); diff --git a/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsRequestTest.java index c18926dc57d20..3251515492a34 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsRequestTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsRequestTest.java @@ -70,13 +70,13 @@ public void testPartitionDir() { .setPartitions(asList(0, 1)), new AlterReplicaLogDirTopic() .setName("topic2") - .setPartitions(asList(7))).iterator())), + .setPartitions(singletonList(7))).iterator())), new AlterReplicaLogDir() .setPath("/data1") .setTopics(new AlterReplicaLogDirTopicCollection( - asList(new AlterReplicaLogDirTopic() + singletonList(new AlterReplicaLogDirTopic() .setName("topic3") - .setPartitions(asList(12))).iterator()))).iterator())); + .setPartitions(singletonList(12))).iterator()))).iterator())); AlterReplicaLogDirsRequest request = new AlterReplicaLogDirsRequest.Builder(data).build(); Map expect = new HashMap<>(); expect.put(new TopicPartition("topic", 0), "/data0"); diff --git a/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsResponseTest.java b/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsResponseTest.java index edc441cf9c828..347a8c26bc067 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsResponseTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/AlterReplicaLogDirsResponseTest.java @@ -16,6 +16,7 @@ */ package org.apache.kafka.common.requests; +import java.util.Collections; import java.util.Map; import org.apache.kafka.common.message.AlterReplicaLogDirsResponseData; @@ -44,7 +45,7 @@ public void testErrorCounts() { .setErrorCode(Errors.NONE.code()))), new AlterReplicaLogDirTopicResult() .setTopicName("t1") - .setPartitions(asList( + .setPartitions(Collections.singletonList( new AlterReplicaLogDirPartitionResult() .setPartitionIndex(0) .setErrorCode(Errors.LOG_DIR_NOT_FOUND.code()))))); diff --git a/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsResponseTest.java b/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsResponseTest.java index 3baf3af3f26ea..339ef9be4a5df 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsResponseTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsResponseTest.java @@ -80,7 +80,7 @@ public class DeleteAclsResponseTest { private static final DeleteAclsFilterResult PREFIXED_RESPONSE = new DeleteAclsFilterResult().setMatchingAcls(asList( LITERAL_ACL1, PREFIXED_ACL1)); - private static final DeleteAclsFilterResult UNKNOWN_RESPONSE = new DeleteAclsFilterResult().setMatchingAcls(asList( + private static final DeleteAclsFilterResult UNKNOWN_RESPONSE = new DeleteAclsFilterResult().setMatchingAcls(singletonList( UNKNOWN_ACL)); @Test diff --git a/clients/src/test/java/org/apache/kafka/common/requests/LeaderAndIsrRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/LeaderAndIsrRequestTest.java index 83c33e4903fba..ce3e1548d31d8 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/LeaderAndIsrRequestTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/LeaderAndIsrRequestTest.java @@ -118,8 +118,8 @@ public void testVersionLogic() { .setIsr(asList(0, 1)) .setPartitionEpoch(10) .setReplicas(asList(0, 1, 2)) - .setAddingReplicas(asList(3)) - .setRemovingReplicas(asList(2)), + .setAddingReplicas(Collections.singletonList(3)) + .setRemovingReplicas(Collections.singletonList(2)), new LeaderAndIsrPartitionState() .setTopicName("topic0") .setPartitionIndex(1) diff --git a/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java index 83c4b101d8969..81b8b8090e56f 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java @@ -60,12 +60,12 @@ public void testDuplicatePartitions() { @Test public void testGetErrorResponse() { for (short version = 1; version <= ApiKeys.LIST_OFFSETS.latestVersion(); version++) { - List topics = Arrays.asList( + List topics = Collections.singletonList( new ListOffsetsTopic() - .setName("topic") - .setPartitions(Collections.singletonList( - new ListOffsetsPartition() - .setPartitionIndex(0)))); + .setName("topic") + .setPartitions(Collections.singletonList( + new ListOffsetsPartition() + .setPartitionIndex(0)))); ListOffsetsRequest request = ListOffsetsRequest.Builder .forConsumer(true, IsolationLevel.READ_COMMITTED, false) .setTargetTimes(topics) @@ -93,12 +93,12 @@ public void testGetErrorResponse() { @Test public void testGetErrorResponseV0() { - List topics = Arrays.asList( + List topics = Collections.singletonList( new ListOffsetsTopic() - .setName("topic") - .setPartitions(Collections.singletonList( - new ListOffsetsPartition() - .setPartitionIndex(0)))); + .setName("topic") + .setPartitions(Collections.singletonList( + new ListOffsetsPartition() + .setPartitionIndex(0)))); ListOffsetsRequest request = ListOffsetsRequest.Builder .forConsumer(true, IsolationLevel.READ_UNCOMMITTED, false) .setTargetTimes(topics) diff --git a/clients/src/test/java/org/apache/kafka/common/requests/RequestResponseTest.java b/clients/src/test/java/org/apache/kafka/common/requests/RequestResponseTest.java index 512a7cea76681..ff862a5bae0b6 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/RequestResponseTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/RequestResponseTest.java @@ -1198,25 +1198,25 @@ private AssignReplicasToDirsRequest createAssignReplicasToDirsRequest(short vers .setDirectories(Arrays.asList( new AssignReplicasToDirsRequestData.DirectoryData() .setId(Uuid.randomUuid()) - .setTopics(Arrays.asList( + .setTopics(singletonList( new AssignReplicasToDirsRequestData.TopicData() .setTopicId(Uuid.fromString("qo0Pcp70TdGnAa7YKMKCqw")) - .setPartitions(Arrays.asList( + .setPartitions(singletonList( new AssignReplicasToDirsRequestData.PartitionData() .setPartitionIndex(8) )) )), new AssignReplicasToDirsRequestData.DirectoryData() .setId(Uuid.randomUuid()) - .setTopics(Arrays.asList( + .setTopics(singletonList( new AssignReplicasToDirsRequestData.TopicData() .setTopicId(Uuid.fromString("yEu11V7HTRGIwm6FDWFhzg")) - .setPartitions(Arrays.asList( + .setPartitions(asList( new AssignReplicasToDirsRequestData.PartitionData() .setPartitionIndex(2), new AssignReplicasToDirsRequestData.PartitionData() .setPartitionIndex(80) - )) + )) )) )); return new AssignReplicasToDirsRequest.Builder(data).build(version); @@ -1229,10 +1229,10 @@ private AssignReplicasToDirsResponse createAssignReplicasToDirsResponse() { .setDirectories(Arrays.asList( new AssignReplicasToDirsResponseData.DirectoryData() .setId(Uuid.randomUuid()) - .setTopics(Arrays.asList( + .setTopics(singletonList( new AssignReplicasToDirsResponseData.TopicData() .setTopicId(Uuid.fromString("sKhZV8LnTA275KvByB9bVg")) - .setPartitions(Arrays.asList( + .setPartitions(singletonList( new AssignReplicasToDirsResponseData.PartitionData() .setPartitionIndex(8) .setErrorCode(Errors.NONE.code()) @@ -1240,10 +1240,10 @@ private AssignReplicasToDirsResponse createAssignReplicasToDirsResponse() { )), new AssignReplicasToDirsResponseData.DirectoryData() .setId(Uuid.randomUuid()) - .setTopics(Arrays.asList( + .setTopics(singletonList( new AssignReplicasToDirsResponseData.TopicData() .setTopicId(Uuid.fromString("ORLP5NEzRo64SvKq1hIVQg")) - .setPartitions(Arrays.asList( + .setPartitions(asList( new AssignReplicasToDirsResponseData.PartitionData() .setPartitionIndex(2) .setErrorCode(Errors.UNKNOWN_TOPIC_OR_PARTITION.code()), @@ -1258,7 +1258,7 @@ private AssignReplicasToDirsResponse createAssignReplicasToDirsResponse() { private DescribeTopicPartitionsRequest createDescribeTopicPartitionsRequest(short version) { DescribeTopicPartitionsRequestData data = new DescribeTopicPartitionsRequestData() - .setTopics(Arrays.asList(new DescribeTopicPartitionsRequestData.TopicRequest().setName("foo"))) + .setTopics(singletonList(new DescribeTopicPartitionsRequestData.TopicRequest().setName("foo"))) .setCursor(new DescribeTopicPartitionsRequestData.Cursor().setTopicName("foo").setPartitionIndex(1)); return new DescribeTopicPartitionsRequest.Builder(data).build(version); } @@ -1273,13 +1273,13 @@ private DescribeTopicPartitionsResponse createDescribeTopicPartitionsResponse() .setIsInternal(false) .setName("foo") .setTopicAuthorizedOperations(0) - .setPartitions(Arrays.asList( + .setPartitions(singletonList( new DescribeTopicPartitionsResponseData.DescribeTopicPartitionsResponsePartition() .setErrorCode((short) 0) - .setIsrNodes(Arrays.asList(1)) + .setIsrNodes(singletonList(1)) .setPartitionIndex(1) .setLeaderId(1) - .setReplicaNodes(Arrays.asList(1)) + .setReplicaNodes(singletonList(1)) .setLeaderEpoch(0) )) ); @@ -1336,7 +1336,7 @@ private ControllerRegistrationRequest createControllerRegistrationRequest(short setIncarnationId(Uuid.fromString("qiTdnbu6RPazh1Aufq4dxw")). setZkMigrationReady(true). setFeatures(new ControllerRegistrationRequestData.FeatureCollection( - Arrays.asList( + singletonList( new ControllerRegistrationRequestData.Feature(). setName("metadata.version"). setMinSupportedVersion((short) 1). @@ -1344,7 +1344,7 @@ private ControllerRegistrationRequest createControllerRegistrationRequest(short ).iterator() )). setListeners(new ControllerRegistrationRequestData.ListenerCollection( - Arrays.asList( + singletonList( new ControllerRegistrationRequestData.Listener(). setName("CONTROLLER"). setName("localhost"). @@ -3543,7 +3543,7 @@ private BrokerRegistrationRequest createBrokerRegistrationRequest(short v) { .setListeners(new BrokerRegistrationRequestData.ListenerCollection(singletonList( new BrokerRegistrationRequestData.Listener()).iterator())) .setIncarnationId(Uuid.randomUuid()) - .setLogDirs(Arrays.asList(Uuid.fromString("qaJjNJ05Q36kEgeTBDcj0Q"))) + .setLogDirs(singletonList(Uuid.fromString("qaJjNJ05Q36kEgeTBDcj0Q"))) .setPreviousBrokerEpoch(123L); return new BrokerRegistrationRequest.Builder(data).build(v); } diff --git a/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java index 2dd17f776ec95..a7d8df35871a2 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java @@ -92,7 +92,7 @@ public void testVersionLogic() { .setIsr(asList(0, 1)) .setZkVersion(10) .setReplicas(asList(0, 1, 2)) - .setOfflineReplicas(asList(2)), + .setOfflineReplicas(Collections.singletonList(2)), new UpdateMetadataPartitionState() .setTopicName(topic0) .setPartitionIndex(1) @@ -143,12 +143,12 @@ public void testVersionLogic() { .setEndpoints(broker0Endpoints), new UpdateMetadataBroker() .setId(1) - .setEndpoints(asList( - new UpdateMetadataEndpoint() - .setHost("host1") - .setPort(9090) - .setSecurityProtocol(SecurityProtocol.PLAINTEXT.id) - .setListener("PLAINTEXT") + .setEndpoints(Collections.singletonList( + new UpdateMetadataEndpoint() + .setHost("host1") + .setPort(9090) + .setSecurityProtocol(SecurityProtocol.PLAINTEXT.id) + .setListener("PLAINTEXT") )) ); diff --git a/clients/src/test/java/org/apache/kafka/common/security/authenticator/ClientAuthenticationFailureTest.java b/clients/src/test/java/org/apache/kafka/common/security/authenticator/ClientAuthenticationFailureTest.java index d705c75ab3fe9..a0e22ee150552 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/authenticator/ClientAuthenticationFailureTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/authenticator/ClientAuthenticationFailureTest.java @@ -43,7 +43,6 @@ import org.junit.jupiter.api.Test; import java.time.Duration; -import java.util.Arrays; import java.util.Collections; import java.util.HashMap; import java.util.Map; @@ -65,13 +64,13 @@ public void setup() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; saslServerConfigs = new HashMap<>(); - saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Arrays.asList("PLAIN")); + saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Collections.singletonList("PLAIN")); saslClientConfigs = new HashMap<>(); saslClientConfigs.put(CommonClientConfigs.SECURITY_PROTOCOL_CONFIG, "SASL_PLAINTEXT"); saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "PLAIN"); - TestJaasConfig testJaasConfig = TestJaasConfig.createConfiguration("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig testJaasConfig = TestJaasConfig.createConfiguration("PLAIN", Collections.singletonList("PLAIN")); testJaasConfig.setClientOptions("PLAIN", TestJaasConfig.USERNAME, "anotherpassword"); server = createEchoServer(securityProtocol); } diff --git a/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorFailureDelayTest.java b/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorFailureDelayTest.java index 477c6283bb498..87c8b456bcd07 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorFailureDelayTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorFailureDelayTest.java @@ -39,7 +39,6 @@ import java.io.IOException; import java.net.InetSocketAddress; -import java.util.Arrays; import java.util.Collections; import java.util.List; import java.util.Map; @@ -93,7 +92,7 @@ public void teardown() throws Exception { public void testInvalidPasswordSaslPlain() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.setClientOptions("PLAIN", TestJaasConfig.USERNAME, "invalidpassword"); server = createEchoServer(securityProtocol); @@ -141,7 +140,7 @@ public void testDisabledSaslMechanism() throws Exception { public void testClientConnectionClose() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.setClientOptions("PLAIN", TestJaasConfig.USERNAME, "invalidpassword"); server = createEchoServer(securityProtocol); diff --git a/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorTest.java b/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorTest.java index 7063bbe698fff..1ab864f912266 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/authenticator/SaslAuthenticatorTest.java @@ -191,7 +191,7 @@ public void teardown() throws Exception { public void testValidSaslPlainOverSsl() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); checkAuthenticationAndReauthentication(securityProtocol, node); @@ -205,7 +205,7 @@ public void testValidSaslPlainOverSsl() throws Exception { public void testValidSaslPlainOverPlaintext() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); checkAuthenticationAndReauthentication(securityProtocol, node); @@ -264,7 +264,7 @@ public void testSaslAuthenticationMaxReceiveSize() throws Exception { public void testInvalidPasswordSaslPlain() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.setClientOptions("PLAIN", TestJaasConfig.USERNAME, "invalidpassword"); server = createEchoServer(securityProtocol); @@ -281,7 +281,7 @@ public void testInvalidPasswordSaslPlain() throws Exception { public void testInvalidUsernameSaslPlain() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.setClientOptions("PLAIN", "invaliduser", TestJaasConfig.PASSWORD); server = createEchoServer(securityProtocol); @@ -297,7 +297,7 @@ public void testInvalidUsernameSaslPlain() throws Exception { @Test public void testMissingUsernameSaslPlain() throws Exception { String node = "0"; - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.setClientOptions("PLAIN", null, "mypassword"); SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; @@ -321,7 +321,7 @@ public void testMissingUsernameSaslPlain() throws Exception { @Test public void testMissingPasswordSaslPlain() throws Exception { String node = "0"; - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.setClientOptions("PLAIN", "myuser", null); SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; @@ -412,7 +412,7 @@ static void reset() { public void testMechanismPluggability() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("DIGEST-MD5", Arrays.asList("DIGEST-MD5")); + configureMechanisms("DIGEST-MD5", Collections.singletonList("DIGEST-MD5")); configureDigestMd5ServerCallback(securityProtocol); server = createEchoServer(securityProtocol); @@ -486,7 +486,7 @@ public void testMultipleServerMechanisms() throws Exception { @Test public void testValidSaslScramSha256() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("SCRAM-SHA-256", Arrays.asList("SCRAM-SHA-256")); + configureMechanisms("SCRAM-SHA-256", Collections.singletonList("SCRAM-SHA-256")); server = createEchoServer(securityProtocol); updateScramCredentialCache(TestJaasConfig.USERNAME, TestJaasConfig.PASSWORD); @@ -516,7 +516,7 @@ public void testValidSaslScramMechanisms() throws Exception { @Test public void testInvalidPasswordSaslScram() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Arrays.asList("SCRAM-SHA-256")); + TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Collections.singletonList("SCRAM-SHA-256")); Map options = new HashMap<>(); options.put("username", TestJaasConfig.USERNAME); options.put("password", "invalidpassword"); @@ -536,7 +536,7 @@ public void testInvalidPasswordSaslScram() throws Exception { @Test public void testUnknownUserSaslScram() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Arrays.asList("SCRAM-SHA-256")); + TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Collections.singletonList("SCRAM-SHA-256")); Map options = new HashMap<>(); options.put("username", "unknownUser"); options.put("password", TestJaasConfig.PASSWORD); @@ -582,7 +582,7 @@ public void testScramUsernameWithSpecialCharacters() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; String username = "special user= test,scram"; String password = username + "-password"; - TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Arrays.asList("SCRAM-SHA-256")); + TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Collections.singletonList("SCRAM-SHA-256")); Map options = new HashMap<>(); options.put("username", username); options.put("password", password); @@ -597,7 +597,7 @@ public void testScramUsernameWithSpecialCharacters() throws Exception { @Test public void testTokenAuthenticationOverSaslScram() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Arrays.asList("SCRAM-SHA-256")); + TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Collections.singletonList("SCRAM-SHA-256")); //create jaas config for token auth Map options = new HashMap<>(); @@ -633,7 +633,7 @@ public void testTokenAuthenticationOverSaslScram() throws Exception { @Test public void testTokenReauthenticationOverSaslScram() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Arrays.asList("SCRAM-SHA-256")); + TestJaasConfig jaasConfig = configureMechanisms("SCRAM-SHA-256", Collections.singletonList("SCRAM-SHA-256")); // create jaas config for token auth Map options = new HashMap<>(); @@ -745,7 +745,7 @@ public void testUnauthenticatedApiVersionsRequestOverSslHandshakeVersion1() thro public void testApiVersionsRequestWithServerUnsupportedVersion() throws Exception { short handshakeVersion = ApiKeys.SASL_HANDSHAKE.latestVersion(); SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send ApiVersionsRequest with unsupported version and validate error response. @@ -787,7 +787,7 @@ public void testApiVersionsRequestWithServerUnsupportedVersion() throws Exceptio */ @Test public void testSaslUnsupportedClientVersions() throws Exception { - configureMechanisms("SCRAM-SHA-512", Arrays.asList("SCRAM-SHA-512")); + configureMechanisms("SCRAM-SHA-512", Collections.singletonList("SCRAM-SHA-512")); server = startServerApiVersionsUnsupportedByClient(SecurityProtocol.SASL_SSL, "SCRAM-SHA-512"); updateScramCredentialCache(TestJaasConfig.USERNAME, TestJaasConfig.PASSWORD); @@ -806,7 +806,7 @@ public void testSaslUnsupportedClientVersions() throws Exception { public void testInvalidApiVersionsRequest() throws Exception { short handshakeVersion = ApiKeys.SASL_HANDSHAKE.latestVersion(); SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send ApiVersionsRequest with invalid version and validate error response. @@ -848,7 +848,7 @@ public void testForBrokenSaslHandshakeVersionBump() { public void testValidApiVersionsRequest() throws Exception { short handshakeVersion = ApiKeys.SASL_HANDSHAKE.latestVersion(); SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send ApiVersionsRequest with valid version and validate error response. @@ -878,7 +878,7 @@ public void testValidApiVersionsRequest() throws Exception { @Test public void testSaslHandshakeRequestWithUnsupportedVersion() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send SaslHandshakeRequest and validate that connection is closed by server. @@ -905,7 +905,7 @@ public void testSaslHandshakeRequestWithUnsupportedVersion() throws Exception { @Test public void testInvalidSaslPacket() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send invalid SASL packet after valid handshake request @@ -944,7 +944,7 @@ public void testInvalidSaslPacket() throws Exception { @Test public void testInvalidApiVersionsRequestSequence() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send handshake request followed by ApiVersionsRequest @@ -970,7 +970,7 @@ public void testInvalidApiVersionsRequestSequence() throws Exception { @Test public void testPacketSizeTooBig() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send SASL packet with large size after valid handshake request @@ -1010,7 +1010,7 @@ public void testPacketSizeTooBig() throws Exception { @Test public void testDisallowedKafkaRequestsBeforeAuthentication() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Send metadata request before Kafka SASL handshake request @@ -1047,7 +1047,7 @@ public void testDisallowedKafkaRequestsBeforeAuthentication() throws Exception { */ @Test public void testInvalidLoginModule() throws Exception { - TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); jaasConfig.createOrUpdateEntry(TestJaasConfig.LOGIN_CONTEXT_CLIENT, "InvalidLoginModule", TestJaasConfig.defaultClientOptions()); SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; @@ -1270,7 +1270,7 @@ public void testServerLoginCallbackOverride() throws Exception { public void testDisabledMechanism() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("PLAIN", Arrays.asList("DIGEST-MD5")); + configureMechanisms("PLAIN", Collections.singletonList("DIGEST-MD5")); server = createEchoServer(securityProtocol); createAndCheckClientConnectionFailure(securityProtocol, node); @@ -1285,7 +1285,7 @@ public void testDisabledMechanism() throws Exception { public void testInvalidMechanism() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "INVALID"); server = createEchoServer(securityProtocol); @@ -1312,7 +1312,7 @@ public void testInvalidMechanism() throws Exception { public void testClientDynamicJaasConfiguration() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "PLAIN"); - saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Arrays.asList("PLAIN")); + saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Collections.singletonList("PLAIN")); Map serverOptions = new HashMap<>(); serverOptions.put("user_user1", "user1-secret"); serverOptions.put("user_user2", "user2-secret"); @@ -1359,7 +1359,7 @@ public void testClientDynamicJaasConfiguration() throws Exception { public void testServerDynamicJaasConfiguration() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "PLAIN"); - saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Arrays.asList("PLAIN")); + saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Collections.singletonList("PLAIN")); Map serverOptions = new HashMap<>(); serverOptions.put("user_user1", "user1-secret"); serverOptions.put("user_user2", "user2-secret"); @@ -1385,7 +1385,7 @@ public void testServerDynamicJaasConfiguration() throws Exception { public void testJaasConfigurationForListener() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT; saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "PLAIN"); - saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Arrays.asList("PLAIN")); + saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Collections.singletonList("PLAIN")); TestJaasConfig staticJaasConfig = new TestJaasConfig(); @@ -1573,7 +1573,7 @@ public void oldSaslScramSslClientWithoutSaslAuthenticateHeaderFailure() throws E public void testValidSaslOauthBearerMechanism() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("OAUTHBEARER", Arrays.asList("OAUTHBEARER")); + configureMechanisms("OAUTHBEARER", Collections.singletonList("OAUTHBEARER")); server = createEchoServer(securityProtocol); createAndCheckClientConnection(securityProtocol, node); } @@ -1588,7 +1588,7 @@ public void testCannotReauthenticateWithDifferentPrincipal() throws Exception { saslClientConfigs.put(SaslConfigs.SASL_LOGIN_CALLBACK_HANDLER_CLASS, AlternateLoginCallbackHandler.class.getName()); configureMechanisms(OAuthBearerLoginModule.OAUTHBEARER_MECHANISM, - Arrays.asList(OAuthBearerLoginModule.OAUTHBEARER_MECHANISM)); + Collections.singletonList(OAuthBearerLoginModule.OAUTHBEARER_MECHANISM)); server = createEchoServer(securityProtocol); // initial authentication must succeed createClientConnection(securityProtocol, node); @@ -1701,7 +1701,7 @@ public void testCannotReauthenticateAgainFasterThanOneSecond() throws Exception time = new MockTime(); SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; configureMechanisms(OAuthBearerLoginModule.OAUTHBEARER_MECHANISM, - Arrays.asList(OAuthBearerLoginModule.OAUTHBEARER_MECHANISM)); + Collections.singletonList(OAuthBearerLoginModule.OAUTHBEARER_MECHANISM)); server = createEchoServer(securityProtocol); try { createClientConnection(securityProtocol, node); @@ -1748,7 +1748,7 @@ public void testCannotReauthenticateAgainFasterThanOneSecond() throws Exception public void testRepeatedValidSaslPlainOverSsl() throws Exception { String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); /* * Make sure 85% of this value is at least 1 second otherwise it is possible for * the client to start re-authenticating but the server does not start due to @@ -1785,7 +1785,7 @@ public void testValidSaslOauthBearerMechanismWithoutServerTokens() throws Except String node = "0"; SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "OAUTHBEARER"); - saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Arrays.asList("OAUTHBEARER")); + saslServerConfigs.put(BrokerSecurityConfigs.SASL_ENABLED_MECHANISMS_CONFIG, Collections.singletonList("OAUTHBEARER")); saslClientConfigs.put(SaslConfigs.SASL_JAAS_CONFIG, TestJaasConfig.jaasConfigProperty("OAUTHBEARER", Collections.singletonMap("unsecuredLoginStringClaim_sub", TestJaasConfig.USERNAME))); saslServerConfigs.put("listener.name.sasl_ssl.oauthbearer." + SaslConfigs.SASL_JAAS_CONFIG, @@ -1818,7 +1818,7 @@ public void testValidSaslOauthBearerMechanismWithoutServerTokens() throws Except @Test public void testInsufficientScopeSaslOauthBearerMechanism() throws Exception { SecurityProtocol securityProtocol = SecurityProtocol.SASL_SSL; - TestJaasConfig jaasConfig = configureMechanisms("OAUTHBEARER", Arrays.asList("OAUTHBEARER")); + TestJaasConfig jaasConfig = configureMechanisms("OAUTHBEARER", Collections.singletonList("OAUTHBEARER")); // now update the server side to require a scope the client does not provide Map serverJaasConfigOptionsMap = TestJaasConfig.defaultServerOptions("OAUTHBEARER"); serverJaasConfigOptionsMap.put("unsecuredValidatorRequiredScope", "LOGIN_TO_KAFKA"); // causes the failure @@ -1900,7 +1900,7 @@ private void removeClientSslKeystore() { private void verifySaslAuthenticateHeaderInterop(boolean enableHeaderOnServer, boolean enableHeaderOnClient, SecurityProtocol securityProtocol, String saslMechanism) throws Exception { - configureMechanisms(saslMechanism, Arrays.asList(saslMechanism)); + configureMechanisms(saslMechanism, Collections.singletonList(saslMechanism)); createServer(securityProtocol, saslMechanism, enableHeaderOnServer); String node = "0"; @@ -1910,7 +1910,7 @@ private void verifySaslAuthenticateHeaderInterop(boolean enableHeaderOnServer, b private void verifySaslAuthenticateHeaderInteropWithFailure(boolean enableHeaderOnServer, boolean enableHeaderOnClient, SecurityProtocol securityProtocol, String saslMechanism) throws Exception { - TestJaasConfig jaasConfig = configureMechanisms(saslMechanism, Arrays.asList(saslMechanism)); + TestJaasConfig jaasConfig = configureMechanisms(saslMechanism, Collections.singletonList(saslMechanism)); jaasConfig.setClientOptions(saslMechanism, TestJaasConfig.USERNAME, "invalidpassword"); createServer(securityProtocol, saslMechanism, enableHeaderOnServer); @@ -1947,7 +1947,7 @@ private NioEchoServer startServerApiVersionsUnsupportedByClient(final SecurityPr boolean isScram = ScramMechanism.isScram(saslMechanism); if (isScram) - ScramCredentialUtils.createCache(credentialCache, Arrays.asList(saslMechanism)); + ScramCredentialUtils.createCache(credentialCache, Collections.singletonList(saslMechanism)); Supplier apiVersionSupplier = () -> { ApiVersionCollection versionCollection = new ApiVersionCollection(2); @@ -1976,7 +1976,7 @@ private NioEchoServer startServerWithoutSaslAuthenticateHeader(final SecurityPro boolean isScram = ScramMechanism.isScram(saslMechanism); if (isScram) - ScramCredentialUtils.createCache(credentialCache, Arrays.asList(saslMechanism)); + ScramCredentialUtils.createCache(credentialCache, Collections.singletonList(saslMechanism)); Supplier apiVersionSupplier = () -> { ApiVersionsResponse defaultApiVersionResponse = TestUtils.defaultApiVersionsResponse( @@ -2090,7 +2090,7 @@ protected void setSaslAuthenticateAndHandshakeVersions(ApiVersionsResponse apiVe * */ private void testUnauthenticatedApiVersionsRequest(SecurityProtocol securityProtocol, short saslHandshakeVersion) throws Exception { - configureMechanisms("PLAIN", Arrays.asList("PLAIN")); + configureMechanisms("PLAIN", Collections.singletonList("PLAIN")); server = createEchoServer(securityProtocol); // Create non-SASL connection to manually authenticate after ApiVersionsRequest @@ -2499,44 +2499,43 @@ public static class AlternateLoginCallbackHandler implements AuthenticateCallbac public void handle(Callback[] callbacks) throws IOException, UnsupportedCallbackException { DELEGATE.handle(callbacks); // now change any returned token to have a different principal name - if (callbacks.length > 0) - for (Callback callback : callbacks) { - if (callback instanceof OAuthBearerTokenCallback) { - OAuthBearerTokenCallback oauthBearerTokenCallback = (OAuthBearerTokenCallback) callback; - OAuthBearerToken token = oauthBearerTokenCallback.token(); - if (token != null) { - String changedPrincipalNameToUse = token.principalName() - + String.valueOf(++numInvocations); - String headerJson = "{" + claimOrHeaderJsonText("alg", "none") + "}"; - /* - * Use a short lifetime so the background refresh thread replaces it before we - * re-authenticate - */ - String lifetimeSecondsValueToUse = "1"; - String claimsJson; - try { - claimsJson = String.format("{%s,%s,%s}", - expClaimText(Long.parseLong(lifetimeSecondsValueToUse)), - claimOrHeaderJsonText("iat", time.milliseconds() / 1000.0), - claimOrHeaderJsonText("sub", changedPrincipalNameToUse)); - } catch (NumberFormatException e) { - throw new OAuthBearerConfigException(e.getMessage()); - } - try { - Encoder urlEncoderNoPadding = Base64.getUrlEncoder().withoutPadding(); - OAuthBearerUnsecuredJws jws = new OAuthBearerUnsecuredJws(String.format("%s.%s.", - urlEncoderNoPadding.encodeToString(headerJson.getBytes(StandardCharsets.UTF_8)), - urlEncoderNoPadding - .encodeToString(claimsJson.getBytes(StandardCharsets.UTF_8))), - "sub", "scope"); - oauthBearerTokenCallback.token(jws); - } catch (OAuthBearerIllegalTokenException e) { - // occurs if the principal claim doesn't exist or has an empty value - throw new OAuthBearerConfigException(e.getMessage(), e); - } + for (Callback callback : callbacks) { + if (callback instanceof OAuthBearerTokenCallback) { + OAuthBearerTokenCallback oauthBearerTokenCallback = (OAuthBearerTokenCallback) callback; + OAuthBearerToken token = oauthBearerTokenCallback.token(); + if (token != null) { + String changedPrincipalNameToUse = token.principalName() + + ++numInvocations; + String headerJson = "{" + claimOrHeaderJsonText("alg", "none") + "}"; + /* + * Use a short lifetime so the background refresh thread replaces it before we + * re-authenticate + */ + String lifetimeSecondsValueToUse = "1"; + String claimsJson; + try { + claimsJson = String.format("{%s,%s,%s}", + expClaimText(Long.parseLong(lifetimeSecondsValueToUse)), + claimOrHeaderJsonText("iat", time.milliseconds() / 1000.0), + claimOrHeaderJsonText("sub", changedPrincipalNameToUse)); + } catch (NumberFormatException e) { + throw new OAuthBearerConfigException(e.getMessage()); + } + try { + Encoder urlEncoderNoPadding = Base64.getUrlEncoder().withoutPadding(); + OAuthBearerUnsecuredJws jws = new OAuthBearerUnsecuredJws(String.format("%s.%s.", + urlEncoderNoPadding.encodeToString(headerJson.getBytes(StandardCharsets.UTF_8)), + urlEncoderNoPadding + .encodeToString(claimsJson.getBytes(StandardCharsets.UTF_8))), + "sub", "scope"); + oauthBearerTokenCallback.token(jws); + } catch (OAuthBearerIllegalTokenException e) { + // occurs if the principal claim doesn't exist or has an empty value + throw new OAuthBearerConfigException(e.getMessage(), e); } } } + } } private static String claimOrHeaderJsonText(String claimName, String claimValue) { diff --git a/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosNameTest.java b/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosNameTest.java index a6e8f9714dc27..209c11e0a8da7 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosNameTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosNameTest.java @@ -20,6 +20,7 @@ import java.io.IOException; import java.util.Arrays; +import java.util.Collections; import java.util.List; import static org.junit.jupiter.api.Assertions.assertEquals; @@ -117,18 +118,18 @@ public void testToUpperCase() throws Exception { @Test public void testInvalidRules() { - testInvalidRule(Arrays.asList("default")); - testInvalidRule(Arrays.asList("DEFAUL")); - testInvalidRule(Arrays.asList("DEFAULT/L")); - testInvalidRule(Arrays.asList("DEFAULT/g")); - - testInvalidRule(Arrays.asList("rule:[1:$1]")); - testInvalidRule(Arrays.asList("rule:[1:$1]/L/U")); - testInvalidRule(Arrays.asList("rule:[1:$1]/U/L")); - testInvalidRule(Arrays.asList("rule:[1:$1]/LU")); - testInvalidRule(Arrays.asList("RULE:[1:$1/L")); - testInvalidRule(Arrays.asList("RULE:[1:$1]/l")); - testInvalidRule(Arrays.asList("RULE:[2:$1](ABC.*)s/ABC/XYZ/L/g")); + testInvalidRule(Collections.singletonList("default")); + testInvalidRule(Collections.singletonList("DEFAUL")); + testInvalidRule(Collections.singletonList("DEFAULT/L")); + testInvalidRule(Collections.singletonList("DEFAULT/g")); + + testInvalidRule(Collections.singletonList("rule:[1:$1]")); + testInvalidRule(Collections.singletonList("rule:[1:$1]/L/U")); + testInvalidRule(Collections.singletonList("rule:[1:$1]/U/L")); + testInvalidRule(Collections.singletonList("rule:[1:$1]/LU")); + testInvalidRule(Collections.singletonList("RULE:[1:$1/L")); + testInvalidRule(Collections.singletonList("RULE:[1:$1]/l")); + testInvalidRule(Collections.singletonList("RULE:[2:$1](ABC.*)s/ABC/XYZ/L/g")); } private void testInvalidRule(List rules) { diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredLoginCallbackHandlerTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredLoginCallbackHandlerTest.java index e29b7c069c984..4ad4e78b92c1f 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredLoginCallbackHandlerTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredLoginCallbackHandlerTest.java @@ -143,7 +143,7 @@ private static OAuthBearerUnsecuredLoginCallbackHandler createCallbackHandler(Ma OAuthBearerUnsecuredLoginCallbackHandler callbackHandler = new OAuthBearerUnsecuredLoginCallbackHandler(); callbackHandler.time(mockTime); callbackHandler.configure(Collections.emptyMap(), OAuthBearerLoginModule.OAUTHBEARER_MECHANISM, - Arrays.asList(config.getAppConfigurationEntry("KafkaClient")[0])); + Collections.singletonList(config.getAppConfigurationEntry("KafkaClient")[0])); return callbackHandler; } diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredValidatorCallbackHandlerTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredValidatorCallbackHandlerTest.java index d7d6013a45717..4c0d055012988 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredValidatorCallbackHandlerTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredValidatorCallbackHandlerTest.java @@ -21,7 +21,6 @@ import static org.junit.jupiter.api.Assertions.assertNull; import java.nio.charset.StandardCharsets; -import java.util.Arrays; import java.util.Base64; import java.util.Base64.Encoder; import java.util.Collections; @@ -157,7 +156,7 @@ private static OAuthBearerUnsecuredValidatorCallbackHandler createCallbackHandle (Map) options); OAuthBearerUnsecuredValidatorCallbackHandler callbackHandler = new OAuthBearerUnsecuredValidatorCallbackHandler(); callbackHandler.configure(Collections.emptyMap(), OAuthBearerLoginModule.OAUTHBEARER_MECHANISM, - Arrays.asList(config.getAppConfigurationEntry("KafkaClient")[0])); + Collections.singletonList(config.getAppConfigurationEntry("KafkaClient")[0])); return callbackHandler; } diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtilsTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtilsTest.java index ef8997a7bc7a9..c52ed6c4ec1cd 100644 --- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtilsTest.java +++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtilsTest.java @@ -161,14 +161,14 @@ public void validateScope() { long nowMs = TIME.milliseconds(); double nowClaimValue = ((double) nowMs) / 1000; final List noScope = Collections.emptyList(); - final List scope1 = Arrays.asList("scope1"); + final List scope1 = Collections.singletonList("scope1"); final List scope1And2 = Arrays.asList("scope1", "scope2"); for (boolean actualScopeExists : new boolean[] {true, false}) { - List scopes = !actualScopeExists ? Arrays.asList((List) null) + List scopes = !actualScopeExists ? Collections.singletonList((List) null) : Arrays.asList(noScope, scope1, scope1And2); for (List actualScope : scopes) { for (boolean requiredScopeExists : new boolean[] {true, false}) { - List requiredScopes = !requiredScopeExists ? Arrays.asList((List) null) + List requiredScopes = !requiredScopeExists ? Collections.singletonList((List) null) : Arrays.asList(noScope, scope1, scope1And2); for (List requiredScope : requiredScopes) { StringBuilder sb = new StringBuilder("{"); diff --git a/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java b/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java index fe02cbe5a96cc..a99a0a384fbdd 100644 --- a/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java @@ -19,6 +19,7 @@ import org.junit.jupiter.api.Test; import java.util.ArrayList; +import java.util.Collections; import java.util.List; import java.util.stream.Collectors; @@ -31,9 +32,9 @@ public class FlattenedIteratorTest { @Test public void testNestedLists() { List> list = asList( - asList("foo", "a", "bc"), - asList("ddddd"), - asList("", "bar2", "baz45")); + asList("foo", "a", "bc"), + Collections.singletonList("ddddd"), + asList("", "bar2", "baz45")); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); @@ -61,7 +62,7 @@ public void testEmptyList() { @Test public void testNestedSingleEmptyList() { - List> list = asList(emptyList()); + List> list = Collections.singletonList(emptyList()); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); @@ -86,9 +87,9 @@ public void testEmptyListFollowedByNonEmpty() { @Test public void testEmptyListInBetweenNonEmpty() { List> list = asList( - asList("aadwdwdw"), - emptyList(), - asList("ee", "aa", "dd")); + Collections.singletonList("aadwdwdw"), + emptyList(), + asList("ee", "aa", "dd")); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); @@ -100,9 +101,9 @@ public void testEmptyListInBetweenNonEmpty() { @Test public void testEmptyListAtTheEnd() { List> list = asList( - asList("ee", "dd"), - asList("e"), - emptyList()); + asList("ee", "dd"), + Collections.singletonList("e"), + emptyList()); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); From 19982a0e4bb20ec207756c6058163d268cc118a4 Mon Sep 17 00:00:00 2001 From: Sanskar Jhajharia Date: Mon, 3 Jun 2024 18:45:51 +0530 Subject: [PATCH 2/3] update PR --- .../unsecured/OAuthBearerValidationUtils.java | 4 ++-- .../org/apache/kafka/common/utils/Utils.java | 1 - .../org/apache/kafka/clients/MetadataTest.java | 10 +++++----- .../requests/UpdateMetadataRequestTest.java | 10 +++++----- .../common/utils/FlattenedIteratorTest.java | 18 +++++++++--------- 5 files changed, 21 insertions(+), 22 deletions(-) diff --git a/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java b/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java index c9eaac41b4545..4dc720ac59771 100644 --- a/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java +++ b/clients/src/main/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerValidationUtils.java @@ -176,8 +176,8 @@ public static OAuthBearerValidationResult validateScope(OAuthBearerToken token, if (!tokenScope.contains(requiredScopeElement)) return OAuthBearerValidationResult.newFailure(String.format( "The provided scope (%s) was missing a required scope (%s). All required scope elements: %s", - tokenScope, requiredScopeElement, requiredScope), - requiredScope.toString(), null); + tokenScope, requiredScopeElement, requiredScope), + requiredScope.toString(), null); } return OAuthBearerValidationResult.newSuccess(); } diff --git a/clients/src/main/java/org/apache/kafka/common/utils/Utils.java b/clients/src/main/java/org/apache/kafka/common/utils/Utils.java index df7f5a50c1dce..ce67fbdb0c73e 100644 --- a/clients/src/main/java/org/apache/kafka/common/utils/Utils.java +++ b/clients/src/main/java/org/apache/kafka/common/utils/Utils.java @@ -371,7 +371,6 @@ public static boolean isEqualConstantTime(char[] first, char[] second) { int j = i < second.length ? i : 0; if (first[i] != second[j]) { matches = false; - break; } } return matches; diff --git a/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java b/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java index d25468cb03018..ac0f5b293bdef 100644 --- a/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java @@ -1135,13 +1135,13 @@ public void testTopicMetadataOnUpdatePartitionLeadership() { // "input-topic" partition 1 leader changes from node 1 to node 2 metadata.updatePartitionLeadership( - Collections.singletonMap( + Collections.singletonMap( tp1, new Metadata.LeaderIdAndEpoch( - Optional.of(2), - Optional.of(3) - )), - Collections.singletonList(node1) + Optional.of(2), + Optional.of(3) + )), + Collections.singletonList(node1) ); assertEquals(2, metadata.fetch().partitionsForTopic(topic).size()); assertEquals(1, metadata.fetch().partition(tp0).leader().id()); diff --git a/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java index a7d8df35871a2..1a909579fbee5 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/UpdateMetadataRequestTest.java @@ -144,11 +144,11 @@ public void testVersionLogic() { new UpdateMetadataBroker() .setId(1) .setEndpoints(Collections.singletonList( - new UpdateMetadataEndpoint() - .setHost("host1") - .setPort(9090) - .setSecurityProtocol(SecurityProtocol.PLAINTEXT.id) - .setListener("PLAINTEXT") + new UpdateMetadataEndpoint() + .setHost("host1") + .setPort(9090) + .setSecurityProtocol(SecurityProtocol.PLAINTEXT.id) + .setListener("PLAINTEXT") )) ); diff --git a/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java b/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java index a99a0a384fbdd..da54c1d98a4ae 100644 --- a/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java +++ b/clients/src/test/java/org/apache/kafka/common/utils/FlattenedIteratorTest.java @@ -32,9 +32,9 @@ public class FlattenedIteratorTest { @Test public void testNestedLists() { List> list = asList( - asList("foo", "a", "bc"), - Collections.singletonList("ddddd"), - asList("", "bar2", "baz45")); + asList("foo", "a", "bc"), + Collections.singletonList("ddddd"), + asList("", "bar2", "baz45")); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); @@ -87,9 +87,9 @@ public void testEmptyListFollowedByNonEmpty() { @Test public void testEmptyListInBetweenNonEmpty() { List> list = asList( - Collections.singletonList("aadwdwdw"), - emptyList(), - asList("ee", "aa", "dd")); + Collections.singletonList("aadwdwdw"), + emptyList(), + asList("ee", "aa", "dd")); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); @@ -101,9 +101,9 @@ public void testEmptyListInBetweenNonEmpty() { @Test public void testEmptyListAtTheEnd() { List> list = asList( - asList("ee", "dd"), - Collections.singletonList("e"), - emptyList()); + asList("ee", "dd"), + Collections.singletonList("e"), + emptyList()); Iterable flattenedIterable = () -> new FlattenedIterator<>(list.iterator(), l -> l.iterator()); List flattened = new ArrayList<>(); From 47af96327f791d7acae2d448774d306cbf4efee3 Mon Sep 17 00:00:00 2001 From: Sanskar Jhajharia Date: Mon, 3 Jun 2024 18:58:41 +0530 Subject: [PATCH 3/3] Address some comments --- .../apache/kafka/clients/MetadataTest.java | 10 ++-- .../clients/admin/KafkaAdminClientTest.java | 52 +++++++++---------- .../internals/CommitRequestManagerTest.java | 24 ++++----- .../internals/ConsumerCoordinatorTest.java | 3 +- .../requests/ListOffsetsRequestTest.java | 16 +++--- 5 files changed, 52 insertions(+), 53 deletions(-) diff --git a/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java b/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java index ac0f5b293bdef..5030ecfef10af 100644 --- a/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/MetadataTest.java @@ -1136,11 +1136,11 @@ public void testTopicMetadataOnUpdatePartitionLeadership() { // "input-topic" partition 1 leader changes from node 1 to node 2 metadata.updatePartitionLeadership( Collections.singletonMap( - tp1, - new Metadata.LeaderIdAndEpoch( - Optional.of(2), - Optional.of(3) - )), + tp1, + new Metadata.LeaderIdAndEpoch( + Optional.of(2), + Optional.of(3) + )), Collections.singletonList(node1) ); assertEquals(2, metadata.fetch().partitionsForTopic(topic).size()); diff --git a/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java b/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java index eb7aea8ae871d..3d73cacbb77a9 100644 --- a/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java @@ -1598,14 +1598,14 @@ public void testDescribeTopicsWithDescribeTopicPartitionsApiErrorHandling() { .setName(topicName0) .setIsInternal(false) .setPartitions(singletonList(new DescribeTopicPartitionsResponsePartition() - .setIsrNodes(singletonList(0)) - .setErrorCode((short) 0) - .setLeaderEpoch(0) - .setLeaderId(0) - .setEligibleLeaderReplicas(singletonList(1)) - .setLastKnownElr(singletonList(2)) - .setPartitionIndex(0) - .setReplicaNodes(asList(0, 1, 2)))) + .setIsrNodes(singletonList(0)) + .setErrorCode((short) 0) + .setLeaderEpoch(0) + .setLeaderId(0) + .setEligibleLeaderReplicas(singletonList(1)) + .setLastKnownElr(singletonList(2)) + .setPartitionIndex(0) + .setReplicaNodes(asList(0, 1, 2)))) ); dataFirstPart.topics().add(new DescribeTopicPartitionsResponseTopic() .setErrorCode((short) 29) @@ -3000,11 +3000,11 @@ public void testListConsumerGroupsWithTypes() throws Exception { new ListGroupsResponse(new ListGroupsResponseData() .setErrorCode(Errors.NONE.code()) .setGroups(singletonList( - new ListGroupsResponseData.ListedGroup() - .setGroupId("group-1") - .setProtocolType(ConsumerProtocol.PROTOCOL_TYPE) - .setGroupState("Stable") - .setGroupType(GroupType.CLASSIC.toString())))), + new ListGroupsResponseData.ListedGroup() + .setGroupId("group-1") + .setProtocolType(ConsumerProtocol.PROTOCOL_TYPE) + .setGroupState("Stable") + .setGroupType(GroupType.CLASSIC.toString())))), env.cluster().nodeById(0)); final ListConsumerGroupsOptions options = new ListConsumerGroupsOptions().inStates(singleton(ConsumerGroupState.STABLE)); @@ -3560,13 +3560,13 @@ public void testDescribeNonConsumerGroups() throws Exception { DescribeGroupsResponseData data = new DescribeGroupsResponseData(); data.groups().add(DescribeGroupsResponse.groupMetadata( - GROUP_ID, - Errors.NONE, - "", - "non-consumer", - "", - emptyList(), - Collections.emptySet())); + GROUP_ID, + Errors.NONE, + "", + "non-consumer", + "", + emptyList(), + Collections.emptySet())); env.kafkaClient().prepareResponse(new DescribeGroupsResponse(data)); @@ -5372,12 +5372,12 @@ public void testListOffsets() throws Exception { pInfos.add(new PartitionInfo("qux", 0, node0, new Node[]{node0}, new Node[]{node0})); final Cluster cluster = new Cluster( - "mockClusterId", - singletonList(node0), - pInfos, - Collections.emptySet(), - Collections.emptySet(), - node0); + "mockClusterId", + singletonList(node0), + pInfos, + Collections.emptySet(), + Collections.emptySet(), + node0); final TopicPartition tp0 = new TopicPartition("foo", 0); final TopicPartition tp1 = new TopicPartition("bar", 0); diff --git a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java index 043cc211da5b4..4e49749c39957 100644 --- a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/CommitRequestManagerTest.java @@ -1336,12 +1336,12 @@ public ClientResponse mockOffsetCommitResponse(String topic, Errors error) { OffsetCommitResponseData responseData = new OffsetCommitResponseData() .setTopics(Collections.singletonList( - new OffsetCommitResponseData.OffsetCommitResponseTopic() - .setName(topic) - .setPartitions(Collections.singletonList( - new OffsetCommitResponseData.OffsetCommitResponsePartition() - .setErrorCode(error.code()) - .setPartitionIndex(partition))))); + new OffsetCommitResponseData.OffsetCommitResponseTopic() + .setName(topic) + .setPartitions(Collections.singletonList( + new OffsetCommitResponseData.OffsetCommitResponsePartition() + .setErrorCode(error.code()) + .setPartitionIndex(partition))))); OffsetCommitResponse response = mock(OffsetCommitResponse.class); when(response.data()).thenReturn(responseData); return new ClientResponse( @@ -1362,12 +1362,12 @@ public ClientResponse mockOffsetCommitResponseDisconnected(String topic, int par NetworkClientDelegate.UnsentRequest unsentRequest) { OffsetCommitResponseData responseData = new OffsetCommitResponseData() .setTopics(Collections.singletonList( - new OffsetCommitResponseData.OffsetCommitResponseTopic() - .setName(topic) - .setPartitions(Collections.singletonList( - new OffsetCommitResponseData.OffsetCommitResponsePartition() - .setErrorCode(Errors.NONE.code()) - .setPartitionIndex(partition))))); + new OffsetCommitResponseData.OffsetCommitResponseTopic() + .setName(topic) + .setPartitions(Collections.singletonList( + new OffsetCommitResponseData.OffsetCommitResponsePartition() + .setErrorCode(Errors.NONE.code()) + .setPartitionIndex(partition))))); OffsetCommitResponse response = mock(OffsetCommitResponse.class); when(response.data()).thenReturn(responseData); return new ClientResponse( diff --git a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java index 541c6a2a04adc..d2bf9b49cb9d3 100644 --- a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java +++ b/clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerCoordinatorTest.java @@ -311,8 +311,7 @@ public void testPerformAssignmentShouldUpdateGroupSubscriptionAfterAssignmentIfN List> capturedTopics = topicsCaptor.getAllValues(); // expected the final group subscribed topics to be updated to "topic1" - Set expectedTopicsGotCalled = new HashSet<>(singletonList(topic1)); - assertEquals(expectedTopicsGotCalled, capturedTopics.get(0)); + assertEquals(Collections.singleton(topic1), capturedTopics.get(0)); } Mockito.clearInvocations(mockSubscriptionState); diff --git a/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java index 81b8b8090e56f..a18619d1712a1 100644 --- a/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java +++ b/clients/src/test/java/org/apache/kafka/common/requests/ListOffsetsRequestTest.java @@ -62,10 +62,10 @@ public void testGetErrorResponse() { for (short version = 1; version <= ApiKeys.LIST_OFFSETS.latestVersion(); version++) { List topics = Collections.singletonList( new ListOffsetsTopic() - .setName("topic") - .setPartitions(Collections.singletonList( - new ListOffsetsPartition() - .setPartitionIndex(0)))); + .setName("topic") + .setPartitions(Collections.singletonList( + new ListOffsetsPartition() + .setPartitionIndex(0)))); ListOffsetsRequest request = ListOffsetsRequest.Builder .forConsumer(true, IsolationLevel.READ_COMMITTED, false) .setTargetTimes(topics) @@ -95,10 +95,10 @@ public void testGetErrorResponse() { public void testGetErrorResponseV0() { List topics = Collections.singletonList( new ListOffsetsTopic() - .setName("topic") - .setPartitions(Collections.singletonList( - new ListOffsetsPartition() - .setPartitionIndex(0)))); + .setName("topic") + .setPartitions(Collections.singletonList( + new ListOffsetsPartition() + .setPartitionIndex(0)))); ListOffsetsRequest request = ListOffsetsRequest.Builder .forConsumer(true, IsolationLevel.READ_UNCOMMITTED, false) .setTargetTimes(topics)