clientPro
* > supported_versions: Lists which versions of TLS the client supports. In particular, if the client
* > requests TLS 1.3, then the client version field has the value TLSv1.2 and this extension
* > contains the value TLSv1.3; if the client requests TLS 1.2, then the client version field has the
- * > value TLSv1.2 and this extension either doesn’t exist or contains the value TLSv1.2 but not the value TLSv1.3.
+ * > value TLSv1.2 and this extension either doesn't exist or contains the value TLSv1.2 but not the value TLSv1.3.
*
*
* This mean that TLSv1.3 client can fallback to TLSv1.2 but TLSv1.2 client can't change protocol to TLSv1.3.
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 05640f26b7388..4795798908ff8 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
@@ -101,7 +101,7 @@ public void testBinaryNode() throws IOException {
JsonNode textNode = mapper.readTree(writer.toString());
- assertTrue(textNode.isTextual(), String.format("Expected a JSON string but was: %s", textNode.toString()));
+ assertTrue(textNode.isTextual(), String.format("Expected a JSON string but was: %s", textNode));
byte[] actual = MessageUtil.jsonNodeToBinary(textNode, "Test base64 JSON string");
assertArrayEquals(expected, actual);
}
diff --git a/clients/src/test/java/org/apache/kafka/common/record/CompressionRatioEstimatorTest.java b/clients/src/test/java/org/apache/kafka/common/record/CompressionRatioEstimatorTest.java
index 7ba51dbdd0459..879530934fba8 100644
--- a/clients/src/test/java/org/apache/kafka/common/record/CompressionRatioEstimatorTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/record/CompressionRatioEstimatorTest.java
@@ -26,8 +26,8 @@ public class CompressionRatioEstimatorTest {
@Test
public void testUpdateEstimation() {
class EstimationsObservedRatios {
- float currentEstimation;
- float observedRatio;
+ final float currentEstimation;
+ final float observedRatio;
EstimationsObservedRatios(float currentEstimation, float observedRatio) {
this.currentEstimation = currentEstimation;
this.observedRatio = observedRatio;
diff --git a/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordBatchTest.java b/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordBatchTest.java
index f9d3ff3d57fd8..9475b29fe6bcd 100644
--- a/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordBatchTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordBatchTest.java
@@ -187,7 +187,7 @@ public void buildDefaultRecordBatchWithSequenceWrapAround() {
public void testSizeInBytes() {
Header[] headers = new Header[] {
new RecordHeader("foo", "value".getBytes()),
- new RecordHeader("bar", (byte[]) null)
+ new RecordHeader("bar", null)
};
long timestamp = System.currentTimeMillis();
diff --git a/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordTest.java b/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordTest.java
index 20502868a5b66..f171bb9c81e09 100644
--- a/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/record/DefaultRecordTest.java
@@ -39,7 +39,7 @@ public class DefaultRecordTest {
public void testBasicSerde() throws IOException {
Header[] headers = new Header[] {
new RecordHeader("foo", "value".getBytes()),
- new RecordHeader("bar", (byte[]) null),
+ new RecordHeader("bar", null),
new RecordHeader("\"A\\u00ea\\u00f1\\u00fcC\"", "value".getBytes())
};
diff --git a/clients/src/test/java/org/apache/kafka/common/record/FileRecordsTest.java b/clients/src/test/java/org/apache/kafka/common/record/FileRecordsTest.java
index 74b7e2ff137e7..5c09e4dfc13ef 100644
--- a/clients/src/test/java/org/apache/kafka/common/record/FileRecordsTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/record/FileRecordsTest.java
@@ -65,7 +65,7 @@
public class FileRecordsTest {
- private byte[][] values = new byte[][] {
+ private final byte[][] values = new byte[][] {
"abcd".getBytes(),
"efgh".getBytes(),
"ijkl".getBytes()
diff --git a/clients/src/test/java/org/apache/kafka/common/record/UnalignedFileRecordsTest.java b/clients/src/test/java/org/apache/kafka/common/record/UnalignedFileRecordsTest.java
index 9a05a22ca5dcc..859b5a03dedd1 100644
--- a/clients/src/test/java/org/apache/kafka/common/record/UnalignedFileRecordsTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/record/UnalignedFileRecordsTest.java
@@ -31,7 +31,7 @@
public class UnalignedFileRecordsTest {
- private byte[][] values = new byte[][] {
+ private final byte[][] values = new byte[][] {
"foo".getBytes(),
"bar".getBytes()
};
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/AddPartitionsToTxnRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/AddPartitionsToTxnRequestTest.java
index 92bb8741be09d..cac73f7244066 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/AddPartitionsToTxnRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/AddPartitionsToTxnRequestTest.java
@@ -39,13 +39,13 @@
import static org.junit.jupiter.api.Assertions.assertEquals;
public class AddPartitionsToTxnRequestTest {
+ private static final int PRODUCER_ID = 10;
+ private static final short PRODUCER_EPOCH = 1;
+ private static final int THROTTLE_TIME_MS = 10;
+ private static final TopicPartition TP_0 = new TopicPartition("topic", 0);
+ private static final TopicPartition TP_1 = new TopicPartition("topic", 1);
private final String transactionalId1 = "transaction1";
private final String transactionalId2 = "transaction2";
- private static int producerId = 10;
- private static short producerEpoch = 1;
- private static int throttleTimeMs = 10;
- private static TopicPartition tp0 = new TopicPartition("topic", 0);
- private static TopicPartition tp1 = new TopicPartition("topic", 1);
@ParameterizedTest
@ApiKeyVersionsSource(apiKey = ApiKeys.ADD_PARTITIONS_TO_TXN)
@@ -55,15 +55,15 @@ public void testConstructor(short version) {
if (version < 4) {
List partitions = new ArrayList<>();
- partitions.add(tp0);
- partitions.add(tp1);
+ partitions.add(TP_0);
+ partitions.add(TP_1);
- AddPartitionsToTxnRequest.Builder builder = AddPartitionsToTxnRequest.Builder.forClient(transactionalId1, producerId, producerEpoch, partitions);
+ AddPartitionsToTxnRequest.Builder builder = AddPartitionsToTxnRequest.Builder.forClient(transactionalId1, PRODUCER_ID, PRODUCER_EPOCH, partitions);
request = builder.build(version);
assertEquals(transactionalId1, request.data().v3AndBelowTransactionalId());
- assertEquals(producerId, request.data().v3AndBelowProducerId());
- assertEquals(producerEpoch, request.data().v3AndBelowProducerEpoch());
+ assertEquals(PRODUCER_ID, request.data().v3AndBelowProducerId());
+ assertEquals(PRODUCER_EPOCH, request.data().v3AndBelowProducerEpoch());
assertEquals(partitions, AddPartitionsToTxnRequest.getPartitions(request.data().v3AndBelowTopics()));
} else {
AddPartitionsToTxnTransactionCollection transactions = createTwoTransactionCollection();
@@ -77,9 +77,9 @@ public void testConstructor(short version) {
assertEquals(transactions.find(transactionalId1), reqTxn1);
assertEquals(transactions.find(transactionalId2), reqTxn2);
}
- AddPartitionsToTxnResponse response = request.getErrorResponse(throttleTimeMs, Errors.UNKNOWN_TOPIC_OR_PARTITION.exception());
+ AddPartitionsToTxnResponse response = request.getErrorResponse(THROTTLE_TIME_MS, Errors.UNKNOWN_TOPIC_OR_PARTITION.exception());
- assertEquals(throttleTimeMs, response.throttleTimeMs());
+ assertEquals(THROTTLE_TIME_MS, response.throttleTimeMs());
if (version >= 4) {
assertEquals(Errors.UNKNOWN_TOPIC_OR_PARTITION.code(), response.data().errorCode());
@@ -98,8 +98,8 @@ public void testBatchedRequests() {
AddPartitionsToTxnRequest request = builder.build(ApiKeys.ADD_PARTITIONS_TO_TXN.latestVersion());
Map> expectedMap = new HashMap<>();
- expectedMap.put(transactionalId1, Collections.singletonList(tp0));
- expectedMap.put(transactionalId2, Collections.singletonList(tp1));
+ expectedMap.put(transactionalId1, Collections.singletonList(TP_0));
+ expectedMap.put(transactionalId2, Collections.singletonList(TP_1));
assertEquals(expectedMap, request.partitionsByTransaction());
@@ -110,50 +110,50 @@ public void testBatchedRequests() {
AddPartitionsToTxnResponse response = new AddPartitionsToTxnResponse(new AddPartitionsToTxnResponseData()
.setResultsByTransaction(results)
- .setThrottleTimeMs(throttleTimeMs));
+ .setThrottleTimeMs(THROTTLE_TIME_MS));
- assertEquals(Collections.singletonMap(tp0, Errors.UNKNOWN_TOPIC_OR_PARTITION), errorsForTransaction(response.getTransactionTopicResults(transactionalId1)));
- assertEquals(Collections.singletonMap(tp1, Errors.TRANSACTIONAL_ID_AUTHORIZATION_FAILED), errorsForTransaction(response.getTransactionTopicResults(transactionalId2)));
+ assertEquals(Collections.singletonMap(TP_0, Errors.UNKNOWN_TOPIC_OR_PARTITION), errorsForTransaction(response.getTransactionTopicResults(transactionalId1)));
+ assertEquals(Collections.singletonMap(TP_1, Errors.TRANSACTIONAL_ID_AUTHORIZATION_FAILED), errorsForTransaction(response.getTransactionTopicResults(transactionalId2)));
}
@Test
public void testNormalizeRequest() {
List partitions = new ArrayList<>();
- partitions.add(tp0);
- partitions.add(tp1);
+ partitions.add(TP_0);
+ partitions.add(TP_1);
- AddPartitionsToTxnRequest.Builder builder = AddPartitionsToTxnRequest.Builder.forClient(transactionalId1, producerId, producerEpoch, partitions);
+ AddPartitionsToTxnRequest.Builder builder = AddPartitionsToTxnRequest.Builder.forClient(transactionalId1, PRODUCER_ID, PRODUCER_EPOCH, partitions);
AddPartitionsToTxnRequest request = builder.build((short) 3);
AddPartitionsToTxnRequest singleton = request.normalizeRequest();
assertEquals(partitions, singleton.partitionsByTransaction().get(transactionalId1));
AddPartitionsToTxnTransaction transaction = singleton.data().transactions().find(transactionalId1);
- assertEquals(producerId, transaction.producerId());
- assertEquals(producerEpoch, transaction.producerEpoch());
+ assertEquals(PRODUCER_ID, transaction.producerId());
+ assertEquals(PRODUCER_EPOCH, transaction.producerEpoch());
}
private AddPartitionsToTxnTransactionCollection createTwoTransactionCollection() {
AddPartitionsToTxnTopicCollection topics0 = new AddPartitionsToTxnTopicCollection();
topics0.add(new AddPartitionsToTxnTopic()
- .setName(tp0.topic())
- .setPartitions(Collections.singletonList(tp0.partition())));
+ .setName(TP_0.topic())
+ .setPartitions(Collections.singletonList(TP_0.partition())));
AddPartitionsToTxnTopicCollection topics1 = new AddPartitionsToTxnTopicCollection();
topics1.add(new AddPartitionsToTxnTopic()
- .setName(tp1.topic())
- .setPartitions(Collections.singletonList(tp1.partition())));
+ .setName(TP_1.topic())
+ .setPartitions(Collections.singletonList(TP_1.partition())));
AddPartitionsToTxnTransactionCollection transactions = new AddPartitionsToTxnTransactionCollection();
transactions.add(new AddPartitionsToTxnTransaction()
.setTransactionalId(transactionalId1)
- .setProducerId(producerId)
- .setProducerEpoch(producerEpoch)
+ .setProducerId(PRODUCER_ID)
+ .setProducerEpoch(PRODUCER_EPOCH)
.setVerifyOnly(true)
.setTopics(topics0));
transactions.add(new AddPartitionsToTxnTransaction()
.setTransactionalId(transactionalId2)
- .setProducerId(producerId + 1)
- .setProducerEpoch((short) (producerEpoch + 1))
+ .setProducerId(PRODUCER_ID + 1)
+ .setProducerEpoch((short) (PRODUCER_EPOCH + 1))
.setVerifyOnly(false)
.setTopics(topics1));
return transactions;
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsRequestTest.java
index e88a5b94aabfc..b2901dabcf551 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/DeleteAclsRequestTest.java
@@ -29,9 +29,9 @@
import org.junit.jupiter.api.Test;
import java.nio.ByteBuffer;
+import java.util.Arrays;
import java.util.stream.Collectors;
-import static java.util.Arrays.asList;
import static org.junit.jupiter.api.Assertions.assertEquals;
import static org.junit.jupiter.api.Assertions.assertThrows;
@@ -110,7 +110,7 @@ private static void assertRequestEquals(final DeleteAclsRequest original, final
}
private static DeleteAclsRequestData requestData(AclBindingFilter... acls) {
- return new DeleteAclsRequestData().setFilters(asList(acls).stream()
+ return new DeleteAclsRequestData().setFilters(Arrays.stream(acls)
.map(DeleteAclsRequest::deleteAclsFilter)
.collect(Collectors.toList()));
}
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/DeleteGroupsResponseTest.java b/clients/src/test/java/org/apache/kafka/common/requests/DeleteGroupsResponseTest.java
index ff352f1ab87b3..7cf7ad8b3371c 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/DeleteGroupsResponseTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/DeleteGroupsResponseTest.java
@@ -34,47 +34,44 @@ public class DeleteGroupsResponseTest {
private static final String GROUP_ID_1 = "groupId1";
private static final String GROUP_ID_2 = "groupId2";
private static final int THROTTLE_TIME_MS = 10;
- private static DeleteGroupsResponse deleteGroupsResponse;
-
- static {
- deleteGroupsResponse = new DeleteGroupsResponse(
- new DeleteGroupsResponseData()
- .setResults(
- new DeletableGroupResultCollection(Arrays.asList(
- new DeletableGroupResult()
- .setGroupId(GROUP_ID_1)
- .setErrorCode(Errors.NONE.code()),
- new DeletableGroupResult()
- .setGroupId(GROUP_ID_2)
- .setErrorCode(Errors.GROUP_AUTHORIZATION_FAILED.code())).iterator()
- )
+ private static final DeleteGroupsResponse DELETE_GROUPS_RESPONSE = new DeleteGroupsResponse(
+ new DeleteGroupsResponseData()
+ .setResults(
+ new DeletableGroupResultCollection(Arrays.asList(
+ new DeletableGroupResult()
+ .setGroupId(GROUP_ID_1)
+ .setErrorCode(Errors.NONE.code()),
+ new DeletableGroupResult()
+ .setGroupId(GROUP_ID_2)
+ .setErrorCode(Errors.GROUP_AUTHORIZATION_FAILED.code())).iterator()
)
- .setThrottleTimeMs(THROTTLE_TIME_MS));
- }
+ )
+ .setThrottleTimeMs(THROTTLE_TIME_MS));
+
@Test
public void testGetErrorWithExistingGroupIds() {
- assertEquals(Errors.NONE, deleteGroupsResponse.get(GROUP_ID_1));
- assertEquals(Errors.GROUP_AUTHORIZATION_FAILED, deleteGroupsResponse.get(GROUP_ID_2));
+ assertEquals(Errors.NONE, DELETE_GROUPS_RESPONSE.get(GROUP_ID_1));
+ assertEquals(Errors.GROUP_AUTHORIZATION_FAILED, DELETE_GROUPS_RESPONSE.get(GROUP_ID_2));
Map expectedErrors = new HashMap<>();
expectedErrors.put(GROUP_ID_1, Errors.NONE);
expectedErrors.put(GROUP_ID_2, Errors.GROUP_AUTHORIZATION_FAILED);
- assertEquals(expectedErrors, deleteGroupsResponse.errors());
+ assertEquals(expectedErrors, DELETE_GROUPS_RESPONSE.errors());
Map expectedErrorCounts = new HashMap<>();
expectedErrorCounts.put(Errors.NONE, 1);
expectedErrorCounts.put(Errors.GROUP_AUTHORIZATION_FAILED, 1);
- assertEquals(expectedErrorCounts, deleteGroupsResponse.errorCounts());
+ assertEquals(expectedErrorCounts, DELETE_GROUPS_RESPONSE.errorCounts());
}
@Test
public void testGetErrorWithInvalidGroupId() {
- assertThrows(IllegalArgumentException.class, () -> deleteGroupsResponse.get("invalid-group-id"));
+ assertThrows(IllegalArgumentException.class, () -> DELETE_GROUPS_RESPONSE.get("invalid-group-id"));
}
@Test
public void testGetThrottleTimeMs() {
- assertEquals(THROTTLE_TIME_MS, deleteGroupsResponse.throttleTimeMs());
+ assertEquals(THROTTLE_TIME_MS, DELETE_GROUPS_RESPONSE.throttleTimeMs());
}
}
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/DescribeAclsRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/DescribeAclsRequestTest.java
index 00ce57a6002e2..9ca0e74d08405 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/DescribeAclsRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/DescribeAclsRequestTest.java
@@ -101,7 +101,7 @@ public void shouldRoundTripAnyV1() {
private static void assertRequestEquals(final DescribeAclsRequest original, final DescribeAclsRequest actual) {
final AclBindingFilter originalFilter = original.filter();
- final AclBindingFilter acttualFilter = actual.filter();
- assertEquals(originalFilter, acttualFilter);
+ final AclBindingFilter actualFilter = actual.filter();
+ assertEquals(originalFilter, actualFilter);
}
}
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupRequestTest.java
index 1694ef5fdf938..e9aa88e9ec521 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupRequestTest.java
@@ -38,9 +38,6 @@ public class LeaveGroupRequestTest {
private final String groupId = "group_id";
private final String memberIdOne = "member_1";
- private final String instanceIdOne = "instance_1";
- private final String memberIdTwo = "member_2";
- private final String instanceIdTwo = "instance_2";
private final int throttleTimeMs = 10;
@@ -51,10 +48,10 @@ public class LeaveGroupRequestTest {
public void setUp() {
members = Arrays.asList(new MemberIdentity()
.setMemberId(memberIdOne)
- .setGroupInstanceId(instanceIdOne),
+ .setGroupInstanceId("instance_1"),
new MemberIdentity()
- .setMemberId(memberIdTwo)
- .setGroupInstanceId(instanceIdTwo));
+ .setMemberId("member_2")
+ .setGroupInstanceId("instance_2"));
builder = new LeaveGroupRequest.Builder(
groupId,
members
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupResponseTest.java b/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupResponseTest.java
index 4bb5c72281952..0f6a2dec820d8 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupResponseTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/LeaveGroupResponseTest.java
@@ -42,11 +42,6 @@
public class LeaveGroupResponseTest {
- private final String memberIdOne = "member_1";
- private final String instanceIdOne = "instance_1";
- private final String memberIdTwo = "member_2";
- private final String instanceIdTwo = "instance_2";
-
private final int throttleTimeMs = 10;
private List memberResponses;
@@ -54,12 +49,12 @@ public class LeaveGroupResponseTest {
@BeforeEach
public void setUp() {
memberResponses = Arrays.asList(new MemberResponse()
- .setMemberId(memberIdOne)
- .setGroupInstanceId(instanceIdOne)
+ .setMemberId("member_1")
+ .setGroupInstanceId("instance_1")
.setErrorCode(Errors.UNKNOWN_MEMBER_ID.code()),
new MemberResponse()
- .setMemberId(memberIdTwo)
- .setGroupInstanceId(instanceIdTwo)
+ .setMemberId("member_2")
+ .setGroupInstanceId("instance_2")
.setErrorCode(Errors.FENCED_INSTANCE_ID.code())
);
}
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/OffsetCommitRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/OffsetCommitRequestTest.java
index 7da2271d97c60..4ce705766f8b1 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/OffsetCommitRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/OffsetCommitRequestTest.java
@@ -55,11 +55,10 @@ public class OffsetCommitRequestTest {
protected static int throttleTimeMs = 10;
private static OffsetCommitRequestData data;
- private static List topics;
@BeforeEach
public void setUp() {
- topics = Arrays.asList(
+ List topics = Arrays.asList(
new OffsetCommitRequestTopic()
.setName(topicOne)
.setPartitions(Collections.singletonList(
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/OffsetFetchRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/OffsetFetchRequestTest.java
index 6ce7dd93fb904..5077b08836df3 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/OffsetFetchRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/OffsetFetchRequestTest.java
@@ -52,7 +52,7 @@ public class OffsetFetchRequestTest {
private final String group3 = "group3";
private final String group4 = "group4";
private final String group5 = "group5";
- private List groups = Arrays.asList(group1, group2, group3, group4, group5);
+ private final List groups = Arrays.asList(group1, group2, group3, group4, group5);
private final List listOfVersionsNonBatchOffsetFetch = Arrays.asList(0, 1, 2, 3, 4, 5, 6, 7);
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/TxnOffsetCommitRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/TxnOffsetCommitRequestTest.java
index 6ebe7b09391d6..d05f3e8781388 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/TxnOffsetCommitRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/TxnOffsetCommitRequestTest.java
@@ -38,11 +38,7 @@
public class TxnOffsetCommitRequestTest extends OffsetCommitRequestTest {
- private static String transactionalId = "transactionalId";
- private static int producerId = 10;
- private static short producerEpoch = 1;
- private static int generationId = 5;
- private static Map offsets = new HashMap<>();
+ private static final Map OFFSETS = new HashMap<>();
private static TxnOffsetCommitRequest.Builder builder;
private static TxnOffsetCommitRequest.Builder builderWithGroupMetadata;
@@ -50,32 +46,36 @@ public class TxnOffsetCommitRequestTest extends OffsetCommitRequestTest {
@Override
public void setUp() {
super.setUp();
- offsets.clear();
- offsets.put(new TopicPartition(topicOne, partitionOne),
+ OFFSETS.clear();
+ OFFSETS.put(new TopicPartition(topicOne, partitionOne),
new CommittedOffset(
offset,
metadata,
Optional.of((int) leaderEpoch)));
- offsets.put(new TopicPartition(topicTwo, partitionTwo),
+ OFFSETS.put(new TopicPartition(topicTwo, partitionTwo),
new CommittedOffset(
offset,
metadata,
Optional.of((int) leaderEpoch)));
+ String transactionalId = "transactionalId";
+ int producerId = 10;
+ short producerEpoch = 1;
builder = new TxnOffsetCommitRequest.Builder(
transactionalId,
groupId,
producerId,
producerEpoch,
- offsets
+ OFFSETS
);
+ int generationId = 5;
builderWithGroupMetadata = new TxnOffsetCommitRequest.Builder(
transactionalId,
groupId,
producerId,
producerEpoch,
- offsets,
+ OFFSETS,
memberId,
generationId,
Optional.of(groupInstanceId)
@@ -118,7 +118,7 @@ public void testConstructor() {
} else {
request = builderWithGroupMetadata.build(version);
}
- assertEquals(offsets, request.offsets());
+ assertEquals(OFFSETS, request.offsets());
assertEquals(expectedTopics, TxnOffsetCommitRequest.getTopics(request.offsets()));
TxnOffsetCommitResponse response =
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersRequestTest.java b/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersRequestTest.java
index 13e8c8cd94035..1a4890965bb8d 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersRequestTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersRequestTest.java
@@ -29,11 +29,11 @@
public class WriteTxnMarkersRequestTest {
- private static long producerId = 10L;
- private static short producerEpoch = 2;
- private static int coordinatorEpoch = 1;
- private static TransactionResult result = TransactionResult.COMMIT;
- private static TopicPartition topicPartition = new TopicPartition("topic", 73);
+ private static final long PRODUCER_ID = 10L;
+ private static final short PRODUCER_EPOCH = 2;
+ private static final int COORDINATOR_EPOCH = 1;
+ private static final TransactionResult RESULT = TransactionResult.COMMIT;
+ private static final TopicPartition TOPIC_PARTITION = new TopicPartition("topic", 73);
protected static int throttleTimeMs = 10;
@@ -43,8 +43,8 @@ public class WriteTxnMarkersRequestTest {
public void setUp() {
markers = Collections.singletonList(
new WriteTxnMarkersRequest.TxnMarkerEntry(
- producerId, producerEpoch, coordinatorEpoch,
- result, Collections.singletonList(topicPartition))
+ PRODUCER_ID, PRODUCER_EPOCH, COORDINATOR_EPOCH,
+ RESULT, Collections.singletonList(TOPIC_PARTITION))
);
}
@@ -55,11 +55,11 @@ public void testConstructor() {
WriteTxnMarkersRequest request = builder.build(version);
assertEquals(1, request.markers().size());
WriteTxnMarkersRequest.TxnMarkerEntry marker = request.markers().get(0);
- assertEquals(producerId, marker.producerId());
- assertEquals(producerEpoch, marker.producerEpoch());
- assertEquals(coordinatorEpoch, marker.coordinatorEpoch());
- assertEquals(result, marker.transactionResult());
- assertEquals(Collections.singletonList(topicPartition), marker.partitions());
+ assertEquals(PRODUCER_ID, marker.producerId());
+ assertEquals(PRODUCER_EPOCH, marker.producerEpoch());
+ assertEquals(COORDINATOR_EPOCH, marker.coordinatorEpoch());
+ assertEquals(RESULT, marker.transactionResult());
+ assertEquals(Collections.singletonList(TOPIC_PARTITION), marker.partitions());
}
}
@@ -72,7 +72,7 @@ public void testGetErrorResponse() {
request.getErrorResponse(throttleTimeMs, Errors.UNKNOWN_PRODUCER_ID.exception());
assertEquals(Collections.singletonMap(
- topicPartition, Errors.UNKNOWN_PRODUCER_ID), errorResponse.errorsByProducerId().get(producerId));
+ TOPIC_PARTITION, Errors.UNKNOWN_PRODUCER_ID), errorResponse.errorsByProducerId().get(PRODUCER_ID));
assertEquals(Collections.singletonMap(Errors.UNKNOWN_PRODUCER_ID, 1), errorResponse.errorCounts());
// Write txn marker has no throttle time defined in response.
assertEquals(0, errorResponse.throttleTimeMs());
diff --git a/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersResponseTest.java b/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersResponseTest.java
index 2a07412d0d9df..9b146ef318fe7 100644
--- a/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersResponseTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/requests/WriteTxnMarkersResponseTest.java
@@ -29,22 +29,22 @@
public class WriteTxnMarkersResponseTest {
- private static long producerIdOne = 1L;
- private static long producerIdTwo = 2L;
+ private static final long PRODUCER_ID_ONE = 1L;
+ private static final long PRODUCER_ID_TWO = 2L;
- private static TopicPartition tp1 = new TopicPartition("topic", 1);
- private static TopicPartition tp2 = new TopicPartition("topic", 2);
+ private static final TopicPartition TP_1 = new TopicPartition("topic", 1);
+ private static final TopicPartition TP_2 = new TopicPartition("topic", 2);
- private static Errors pidOneError = Errors.UNKNOWN_PRODUCER_ID;
- private static Errors pidTwoError = Errors.INVALID_PRODUCER_EPOCH;
+ private static final Errors PID_ONE_ERROR = Errors.UNKNOWN_PRODUCER_ID;
+ private static final Errors PID_TWO_ERROR = Errors.INVALID_PRODUCER_EPOCH;
private static Map> errorMap;
@BeforeEach
public void setUp() {
errorMap = new HashMap<>();
- errorMap.put(producerIdOne, Collections.singletonMap(tp1, pidOneError));
- errorMap.put(producerIdTwo, Collections.singletonMap(tp2, pidTwoError));
+ errorMap.put(PRODUCER_ID_ONE, Collections.singletonMap(TP_1, PID_ONE_ERROR));
+ errorMap.put(PRODUCER_ID_TWO, Collections.singletonMap(TP_2, PID_TWO_ERROR));
}
@Test
@@ -54,7 +54,7 @@ public void testConstructor() {
expectedErrorCounts.put(Errors.INVALID_PRODUCER_EPOCH, 1);
WriteTxnMarkersResponse response = new WriteTxnMarkersResponse(errorMap);
assertEquals(expectedErrorCounts, response.errorCounts());
- assertEquals(Collections.singletonMap(tp1, pidOneError), response.errorsByProducerId().get(producerIdOne));
- assertEquals(Collections.singletonMap(tp2, pidTwoError), response.errorsByProducerId().get(producerIdTwo));
+ assertEquals(Collections.singletonMap(TP_1, PID_ONE_ERROR), response.errorsByProducerId().get(PRODUCER_ID_ONE));
+ assertEquals(Collections.singletonMap(TP_2, PID_TWO_ERROR), response.errorsByProducerId().get(PRODUCER_ID_TWO));
}
}
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 0a0466f6ff1e6..d705c75ab3fe9 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
@@ -52,13 +52,12 @@
import static org.junit.jupiter.api.Assertions.assertThrows;
public class ClientAuthenticationFailureTest {
- private static MockTime time = new MockTime(50);
+ private static final MockTime TIME = new MockTime(50);
private NioEchoServer server;
private Map saslServerConfigs;
private Map saslClientConfigs;
private final String topic = "test";
- private TestJaasConfig testJaasConfig;
@BeforeEach
public void setup() throws Exception {
@@ -72,7 +71,7 @@ public void setup() throws Exception {
saslClientConfigs.put(CommonClientConfigs.SECURITY_PROTOCOL_CONFIG, "SASL_PLAINTEXT");
saslClientConfigs.put(SaslConfigs.SASL_MECHANISM, "PLAIN");
- testJaasConfig = TestJaasConfig.createConfiguration("PLAIN", Arrays.asList("PLAIN"));
+ TestJaasConfig testJaasConfig = TestJaasConfig.createConfiguration("PLAIN", Arrays.asList("PLAIN"));
testJaasConfig.setClientOptions("PLAIN", TestJaasConfig.USERNAME, "anotherpassword");
server = createEchoServer(securityProtocol);
}
@@ -140,6 +139,6 @@ private NioEchoServer createEchoServer(SecurityProtocol securityProtocol) throws
private NioEchoServer createEchoServer(ListenerName listenerName, SecurityProtocol securityProtocol) throws Exception {
return NetworkTestUtils.createEchoServer(listenerName, securityProtocol,
- new TestSecurityConfig(saslServerConfigs), new CredentialCache(), time);
+ new TestSecurityConfig(saslServerConfigs), new CredentialCache(), TIME);
}
}
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 f80170063ecce..477c6283bb498 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
@@ -54,9 +54,6 @@ public abstract class SaslAuthenticatorFailureDelayTest {
private final MockTime time = new MockTime(1);
private NioEchoServer server;
private Selector selector;
- private ChannelBuilder channelBuilder;
- private CertStores serverCertStores;
- private CertStores clientCertStores;
private Map saslClientConfigs;
private Map saslServerConfigs;
private CredentialCache credentialCache;
@@ -70,8 +67,8 @@ public SaslAuthenticatorFailureDelayTest(int failedAuthenticationDelayMs) {
@BeforeEach
public void setup() throws Exception {
LoginManager.closeAll();
- serverCertStores = new CertStores(true, "localhost");
- clientCertStores = new CertStores(false, "localhost");
+ CertStores serverCertStores = new CertStores(true, "localhost");
+ CertStores clientCertStores = new CertStores(false, "localhost");
saslServerConfigs = serverCertStores.getTrustingConfig(clientCertStores);
saslClientConfigs = clientCertStores.getTrustingConfig(serverCertStores);
credentialCache = new CredentialCache();
@@ -203,7 +200,7 @@ private void createSelector(SecurityProtocol securityProtocol, Map saslClientConfigs;
private Map saslServerConfigs;
private CredentialCache credentialCache;
@@ -169,8 +167,8 @@ public class SaslAuthenticatorTest {
public void setup() throws Exception {
LoginManager.closeAll();
time = Time.SYSTEM;
- serverCertStores = new CertStores(true, "localhost");
- clientCertStores = new CertStores(false, "localhost");
+ CertStores serverCertStores = new CertStores(true, "localhost");
+ CertStores clientCertStores = new CertStores(false, "localhost");
saslServerConfigs = serverCertStores.getTrustingConfig(clientCertStores);
saslClientConfigs = clientCertStores.getTrustingConfig(serverCertStores);
credentialCache = new CredentialCache();
@@ -661,10 +659,9 @@ public void testTokenReauthenticationOverSaslScram() throws Exception {
@Override
public TokenInformation token(String tokenId) {
TokenInformation baseTokenInfo = super.token(tokenId);
- long thisLifetimeMs = System.currentTimeMillis() + tokenLifetime.apply(++callNum).longValue();
- TokenInformation retvalTokenInfo = new TokenInformation(baseTokenInfo.tokenId(), baseTokenInfo.owner(),
+ long thisLifetimeMs = System.currentTimeMillis() + tokenLifetime.apply(++callNum);
+ return new TokenInformation(baseTokenInfo.tokenId(), baseTokenInfo.owner(),
baseTokenInfo.renewers(), baseTokenInfo.issueTimestamp(), thisLifetimeMs, thisLifetimeMs);
- return retvalTokenInfo;
}
};
server = createEchoServer(ListenerName.forSecurityProtocol(securityProtocol), securityProtocol, tokenCache);
@@ -1092,7 +1089,7 @@ public void testClientAuthenticateCallbackHandler() throws Exception {
public void testServerAuthenticateCallbackHandler() throws Exception {
SecurityProtocol securityProtocol = SecurityProtocol.SASL_PLAINTEXT;
TestJaasConfig jaasConfig = configureMechanisms("PLAIN", Collections.singletonList("PLAIN"));
- jaasConfig.createOrUpdateEntry(TestJaasConfig.LOGIN_CONTEXT_SERVER, PlainLoginModule.class.getName(), new HashMap());
+ jaasConfig.createOrUpdateEntry(TestJaasConfig.LOGIN_CONTEXT_SERVER, PlainLoginModule.class.getName(), new HashMap<>());
String callbackPrefix = ListenerName.forSecurityProtocol(securityProtocol).saslMechanismConfigPrefix("PLAIN");
saslServerConfigs.put(callbackPrefix + BrokerSecurityConfigs.SASL_SERVER_CALLBACK_HANDLER_CLASS,
TestServerCallbackHandler.class.getName());
@@ -2199,7 +2196,7 @@ private void checkClientConnection(String node) throws Exception {
NetworkTestUtils.checkClientConnection(selector, node, 100, 10);
}
- private void closeClientConnectionIfNecessary() throws Exception {
+ private void closeClientConnectionIfNecessary() {
if (selector != null) {
selector.close();
selector = null;
@@ -2232,8 +2229,7 @@ private ChannelState createAndCheckClientConnectionFailure(SecurityProtocol secu
throws Exception {
try {
createClientConnection(securityProtocol, node);
- ChannelState finalState = NetworkTestUtils.waitForChannelClose(selector, node, ChannelState.State.AUTHENTICATION_FAILED);
- return finalState;
+ return NetworkTestUtils.waitForChannelClose(selector, node, ChannelState.State.AUTHENTICATION_FAILED);
} finally {
closeClientConnectionIfNecessary();
}
diff --git a/clients/src/test/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java b/clients/src/test/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
index f7ad140cb075e..9f119dfdaa78c 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
@@ -39,7 +39,7 @@ public class TestJaasConfig extends Configuration {
static final String USERNAME = "myuser";
static final String PASSWORD = "mypassword";
- private Map entryMap = new HashMap<>();
+ private final Map entryMap = new HashMap<>();
public static TestJaasConfig createConfiguration(String clientMechanism, List serverMechanisms) {
TestJaasConfig config = new TestJaasConfig();
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 2e063968f2e0a..a6e8f9714dc27 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
@@ -24,7 +24,7 @@
import static org.junit.jupiter.api.Assertions.assertEquals;
import static org.junit.jupiter.api.Assertions.assertNull;
-import static org.junit.jupiter.api.Assertions.fail;
+import static org.junit.jupiter.api.Assertions.assertThrows;
public class KerberosNameTest {
@@ -132,10 +132,8 @@ public void testInvalidRules() {
}
private void testInvalidRule(List rules) {
- try {
- KerberosShortNamer.fromUnparsedRules("REALM.COM", rules);
- fail("should have thrown IllegalArgumentException");
- } catch (IllegalArgumentException e) {
- }
+ assertThrows(
+ IllegalArgumentException.class,
+ () -> KerberosShortNamer.fromUnparsedRules("REALM.COM", rules));
}
}
diff --git a/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosRuleTest.java b/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosRuleTest.java
index 9c785c4eae392..101530a05c5f0 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosRuleTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/kerberos/KerberosRuleTest.java
@@ -17,7 +17,7 @@
package org.apache.kafka.common.security.kerberos;
import static org.junit.jupiter.api.Assertions.assertEquals;
-import static org.junit.jupiter.api.Assertions.fail;
+import static org.junit.jupiter.api.Assertions.assertThrows;
import org.junit.jupiter.api.Test;
@@ -34,16 +34,15 @@ public void testReplaceParameters() throws BadFormatString {
assertEquals(KerberosRule.replaceParameters("hello $0", new String[]{"no recursion $1"}), "hello no recursion $1");
// negative test cases
- try {
- KerberosRule.replaceParameters("$0", new String[]{});
- fail("An out-of-bounds parameter number should trigger an exception!");
- } catch (BadFormatString bfs) {
- }
- try {
- KerberosRule.replaceParameters("hello $a", new String[]{"does not matter"});
- fail("A malformed parameter name should trigger an exception!");
- } catch (BadFormatString bfs) {
- }
+ assertThrows(
+ BadFormatString.class,
+ () -> KerberosRule.replaceParameters("$0", new String[]{}),
+ "An out-of-bounds parameter number should trigger an exception!");
+
+ assertThrows(
+ BadFormatString.class,
+ () -> KerberosRule.replaceParameters("hello $a", new String[]{"does not matter"}),
+ "A malformed parameter name should trigger an exception!");
}
}
diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslClientTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslClientTest.java
index 50ed3fd23c19b..b1aaa0b3d44cc 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslClientTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslClientTest.java
@@ -51,8 +51,8 @@ public class OAuthBearerSaslClientTest {
private final String errorMessage = "Error as expected!";
public class ExtensionsCallbackHandler implements AuthenticateCallbackHandler {
+ private final boolean toThrow;
private boolean configured = false;
- private boolean toThrow;
ExtensionsCallbackHandler(boolean toThrow) {
this.toThrow = toThrow;
diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslServerTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslServerTest.java
index b2cd0e88914e5..7ece041e68e1f 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslServerTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/OAuthBearerSaslServerTest.java
@@ -202,11 +202,6 @@ public void illegalToken() throws Exception {
private byte[] clientInitialResponse(String authorizationId)
throws OAuthBearerConfigException, IOException, UnsupportedCallbackException {
- return clientInitialResponse(authorizationId, false);
- }
-
- private byte[] clientInitialResponse(String authorizationId, boolean illegalToken)
- throws OAuthBearerConfigException, IOException, UnsupportedCallbackException {
return clientInitialResponse(authorizationId, false, Collections.emptyMap());
}
diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/expiring/ExpiringCredentialRefreshingLoginTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/expiring/ExpiringCredentialRefreshingLoginTest.java
index cd946fa0eb069..32895e624b753 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/expiring/ExpiringCredentialRefreshingLoginTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/expiring/ExpiringCredentialRefreshingLoginTest.java
@@ -347,7 +347,7 @@ public void testRefresh() throws Exception {
for (int i = 0; i < numExpectedRefreshes; ++i) {
KafkaFutureImpl waiter = waiters.get(i);
assertTrue(waiter.isDone());
- assertEquals((i + 1) * 1000 * 60 * refreshEveryMinutes, waiter.get().longValue() - startMs);
+ assertEquals((i + 1) * 1000 * 60 * refreshEveryMinutes, waiter.get() - startMs);
}
assertFalse(waiters.get(numExpectedRefreshes).isDone());
@@ -438,7 +438,7 @@ public void testRefreshWithExpirationSmallerThanConfiguredBuffers() throws Excep
for (int i = 0; i < numExpectedRefreshes; ++i) {
KafkaFutureImpl waiter = waiters.get(i);
assertTrue(waiter.isDone());
- assertEquals((i + 1) * 1000 * 60 * refreshEveryMinutes, waiter.get().longValue() - startMs);
+ assertEquals((i + 1) * 1000 * 60 * refreshEveryMinutes, waiter.get() - startMs);
}
assertFalse(waiters.get(numExpectedRefreshes).isDone());
@@ -522,7 +522,7 @@ public long getCreateMs() {
for (int i = 0; i < numExpectedRefreshes; ++i) {
KafkaFutureImpl waiter = waiters.get(i);
assertTrue(waiter.isDone());
- assertEquals((i + 1) * 1000 * 60 * refreshEveryMinutes, waiter.get().longValue() - startMs);
+ assertEquals((i + 1) * 1000 * 60 * refreshEveryMinutes, waiter.get() - startMs);
}
assertFalse(waiters.get(numExpectedRefreshes).isDone());
@@ -603,7 +603,7 @@ public void testRefreshWithMinPeriodIntrusion() throws Exception {
KafkaFutureImpl waiter = waiters.get(i);
assertTrue(waiter.isDone());
assertEquals((i + 1) * 1000 * (60 * refreshEveryMinutes + bufferIntrusionSeconds),
- waiter.get().longValue() - startMs);
+ waiter.get() - startMs);
}
assertFalse(waiters.get(numExpectedRefreshes).isDone());
@@ -683,7 +683,7 @@ public void testRefreshWithPreExpirationBufferIntrusion() throws Exception {
KafkaFutureImpl waiter = waiters.get(i);
assertTrue(waiter.isDone());
assertEquals((i + 1) * 1000 * (60 * refreshEveryMinutes - bufferIntrusionSeconds),
- waiter.get().longValue() - startMs);
+ waiter.get() - startMs);
}
assertFalse(waiters.get(numExpectedRefreshes).isDone());
diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/secured/RefreshingHttpsJwksTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/secured/RefreshingHttpsJwksTest.java
index cc19c74e66f70..7e60313bb3043 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/secured/RefreshingHttpsJwksTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/secured/RefreshingHttpsJwksTest.java
@@ -236,7 +236,7 @@ public String getBody() {
* A mock ScheduledExecutorService just for the test. Note that this is not a generally reusable mock as it does not
* implement some interfaces like scheduleWithFixedDelay, etc. And it does not return ScheduledFuture correctly.
*/
- private class MockExecutorService implements MockTime.Listener {
+ private static class MockExecutorService implements MockTime.Listener {
private final MockTime time;
private final TreeMap>>> waiters = new TreeMap<>();
diff --git a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredJwsTest.java b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredJwsTest.java
index af259c64ecaee..aa105685c2f9e 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredJwsTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/oauthbearer/internals/unsecured/OAuthBearerUnsecuredJwsTest.java
@@ -71,8 +71,8 @@ public void validCompactSerialization() {
assertEquals(4, jws.claims().size());
assertEquals(subject, jws.claims().get("sub"));
assertEquals(subject, jws.principalName());
- assertEquals(issuedAt, Number.class.cast(jws.claims().get("iat")).longValue());
- assertEquals(expirationTime, Number.class.cast(jws.claims().get("exp")).longValue());
+ assertEquals(issuedAt, ((Number) jws.claims().get("iat")).longValue());
+ assertEquals(expirationTime, ((Number) jws.claims().get("exp")).longValue());
assertEquals(expirationTime * 1000, jws.lifetimeMs());
assertEquals(scope, jws.claims().get("scope"));
assertEquals(new HashSet<>(scope), jws.scope());
@@ -110,8 +110,8 @@ private static String compactSerialization(String subject, Long issuedAt, Long e
String headerJson = "{\"alg\":\"" + algorithm + "\"}";
String encodedHeader = encoder.encodeToString(headerJson.getBytes(StandardCharsets.UTF_8));
String subjectJson = subject != null ? "\"sub\":\"" + subject + "\"" : null;
- String issuedAtJson = issuedAt != null ? "\"iat\":" + issuedAt.longValue() : null;
- String expirationTimeJson = expirationTime != null ? "\"exp\":" + expirationTime.longValue() : null;
+ String issuedAtJson = issuedAt != null ? "\"iat\":" + issuedAt : null;
+ String expirationTimeJson = expirationTime != null ? "\"exp\":" + expirationTime : null;
String scopeJson = scope != null ? scopeJson(scope) : null;
String claimsJson = claimsJson(subjectJson, issuedAtJson, expirationTimeJson, scopeJson);
String encodedClaims = encoder.encodeToString(claimsJson.getBytes(StandardCharsets.UTF_8));
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 443a1de4b51c2..e29b7c069c984 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
@@ -105,8 +105,8 @@ public void validOptionsWithExplicitOptionValues()
options.put("unsecuredLoginListClaim_" + "emptyList1", "");
options.put("unsecuredLoginListClaim_" + "emptyList2", ",");
options.put("unsecuredLoginNumberClaim_" + "number", "1");
- long lifetmeSeconds = 10000;
- options.put("unsecuredLoginLifetimeSeconds", String.valueOf(lifetmeSeconds));
+ long lifetimeSeconds = 10000;
+ options.put("unsecuredLoginLifetimeSeconds", String.valueOf(lifetimeSeconds));
options.put("unsecuredLoginPrincipalClaimName", principalClaimName);
if (scopeClaimNameOptionValue != null)
options.put("unsecuredLoginScopeClaimName", scopeClaimNameOptionValue);
@@ -120,7 +120,7 @@ public void validOptionsWithExplicitOptionValues()
OAuthBearerUnsecuredJws jws = (OAuthBearerUnsecuredJws) callback.token();
assertNotNull(jws, "create token failed");
long startMs = mockTime.milliseconds();
- confirmCorrectValues(jws, user, startMs, lifetmeSeconds * 1000);
+ confirmCorrectValues(jws, user, startMs, lifetimeSeconds * 1000);
Map claims = jws.claims();
assertEquals(new HashSet<>(Arrays.asList(actualScopeClaimName, principalClaimName, "iat", "exp", "number",
"list", "emptyList1", "emptyList2")), claims.keySet());
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 88241b72a97a2..ef8997a7bc7a9 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
@@ -44,7 +44,7 @@ public void validateClaimForExistenceAndType() throws OAuthBearerIllegalTokenExc
appendCommaJsonText(sb, "sub", "principalName");
if (useErrorValue)
appendCommaJsonText(sb, claimName, 1);
- else if (exists != null && exists.booleanValue())
+ else if (exists)
appendCommaJsonText(sb, claimName, claimName);
sb.append("}");
String compactSerialization = HEADER_COMPACT_SERIALIZATION + Base64.getUrlEncoder().withoutPadding()
@@ -52,7 +52,7 @@ else if (exists != null && exists.booleanValue())
OAuthBearerUnsecuredJws testJwt = new OAuthBearerUnsecuredJws(compactSerialization, "sub", "scope");
OAuthBearerValidationResult result = OAuthBearerValidationUtils
.validateClaimForExistenceAndType(testJwt, required, claimName, String.class);
- if (useErrorValue || required && !exists.booleanValue())
+ if (useErrorValue || required && !exists)
assertTrue(isFailureWithMessageAndNoFailureScope(result));
else
assertTrue(isSuccess(result));
diff --git a/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramMessagesTest.java b/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramMessagesTest.java
index a286085714abd..066458a68910e 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramMessagesTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramMessagesTest.java
@@ -280,7 +280,7 @@ public void invalidServerFinalMessage() {
checkInvalidScramMessage(ServerFinalMessage.class, invalid);
// Invalid server signature
- invalid = String.format("v=1=23");
+ invalid = "v=1=23";
checkInvalidScramMessage(ServerFinalMessage.class, invalid);
// Invalid extensions
diff --git a/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramSaslServerTest.java b/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramSaslServerTest.java
index 7e83b76f6e131..4f1592b84fbf5 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramSaslServerTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/scram/internals/ScramSaslServerTest.java
@@ -36,13 +36,12 @@ public class ScramSaslServerTest {
private static final String USER_A = "userA";
private static final String USER_B = "userB";
- private ScramMechanism mechanism;
private ScramFormatter formatter;
private ScramSaslServer saslServer;
@BeforeEach
public void setUp() throws Exception {
- mechanism = ScramMechanism.SCRAM_SHA_256;
+ ScramMechanism mechanism = ScramMechanism.SCRAM_SHA_256;
formatter = new ScramFormatter(mechanism);
CredentialCache.Cache credentialCache = new CredentialCache().createCache(mechanism.mechanismName(), ScramCredential.class);
credentialCache.put(USER_A, formatter.generateCredential("passwordA", 4096));
diff --git a/clients/src/test/java/org/apache/kafka/common/security/ssl/CommonNameLoggingTrustManagerFactoryWrapperTest.java b/clients/src/test/java/org/apache/kafka/common/security/ssl/CommonNameLoggingTrustManagerFactoryWrapperTest.java
index 8757dfee60b4f..129e383221e86 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/ssl/CommonNameLoggingTrustManagerFactoryWrapperTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/ssl/CommonNameLoggingTrustManagerFactoryWrapperTest.java
@@ -171,7 +171,7 @@ void testNeverExpiringX509Certificate() throws Exception {
assertDoesNotThrow(() -> wrappedCert.checkValidity(dateRecentPast));
} else {
// Cert not valid yet
- Exception origException = assertThrows(CertificateException.class,
+ assertThrows(CertificateException.class,
() -> cert.checkValidity(dateRecentPast));
// The wrappend certificate class does not check dates at all
assertDoesNotThrow(() -> wrappedCert.checkValidity(dateRecentPast));
diff --git a/clients/src/test/java/org/apache/kafka/common/security/ssl/SslPrincipalMapperTest.java b/clients/src/test/java/org/apache/kafka/common/security/ssl/SslPrincipalMapperTest.java
index ff5a018d4bf9f..52e10fd36b99f 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/ssl/SslPrincipalMapperTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/ssl/SslPrincipalMapperTest.java
@@ -19,7 +19,7 @@
import org.junit.jupiter.api.Test;
import static org.junit.jupiter.api.Assertions.assertEquals;
-import static org.junit.jupiter.api.Assertions.fail;
+import static org.junit.jupiter.api.Assertions.assertThrows;
public class SslPrincipalMapperTest {
@@ -56,11 +56,7 @@ public void testInvalidRules() {
}
private void testInvalidRule(String rules) {
- try {
- System.out.println(SslPrincipalMapper.fromRules(rules));
- fail("should have thrown IllegalArgumentException");
- } catch (IllegalArgumentException e) {
- }
+ assertThrows(IllegalArgumentException.class, () -> SslPrincipalMapper.fromRules(rules));
}
@Test
diff --git a/clients/src/test/java/org/apache/kafka/common/security/ssl/mock/TestKeyManagerFactory.java b/clients/src/test/java/org/apache/kafka/common/security/ssl/mock/TestKeyManagerFactory.java
index 7c9c0dc094c61..596b4e9e5aca7 100644
--- a/clients/src/test/java/org/apache/kafka/common/security/ssl/mock/TestKeyManagerFactory.java
+++ b/clients/src/test/java/org/apache/kafka/common/security/ssl/mock/TestKeyManagerFactory.java
@@ -61,8 +61,8 @@ public static class TestKeyManager extends X509ExtendedKeyManager {
public static final String ALIAS = "TestAlias";
private static final String CN = "localhost";
private static final String SIGNATURE_ALGORITHM = "RSA";
- private KeyPair keyPair;
- private X509Certificate certificate;
+ private final KeyPair keyPair;
+ private final X509Certificate certificate;
protected TestKeyManager() {
try {
diff --git a/clients/src/test/java/org/apache/kafka/common/serialization/SerializationTest.java b/clients/src/test/java/org/apache/kafka/common/serialization/SerializationTest.java
index 15c95a126c349..62ea29b7be348 100644
--- a/clients/src/test/java/org/apache/kafka/common/serialization/SerializationTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/serialization/SerializationTest.java
@@ -26,6 +26,7 @@
import java.nio.charset.StandardCharsets;
import java.util.ArrayList;
import java.util.Arrays;
+import java.util.Collections;
import java.util.HashMap;
import java.util.LinkedList;
import java.util.List;
@@ -62,7 +63,7 @@ public class SerializationTest {
}
};
- private class DummyClass {
+ private static class DummyClass {
}
@SuppressWarnings("unchecked")
@@ -147,7 +148,7 @@ public void stringSerdeConfigureThrowsOnUnknownEncoding() {
@SuppressWarnings("unchecked")
@Test
public void listSerdeShouldReturnEmptyCollection() {
- List testData = Arrays.asList();
+ List testData = Collections.emptyList();
Serde> listSerde = Serdes.ListSerde(ArrayList.class, Serdes.Integer());
assertEquals(testData,
listSerde.deserializer().deserialize(topic, listSerde.serializer().serialize(topic, testData)),
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/AbstractIteratorTest.java b/clients/src/test/java/org/apache/kafka/common/utils/AbstractIteratorTest.java
index 5f3ca4ddf0d17..466d2362f32e8 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/AbstractIteratorTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/AbstractIteratorTest.java
@@ -54,7 +54,7 @@ public void testEmptyIterator() {
}
static class ListIterator extends AbstractIterator {
- private List list;
+ private final List list;
private int position = 0;
public ListIterator(List l) {
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/ConfigUtilsTest.java b/clients/src/test/java/org/apache/kafka/common/utils/ConfigUtilsTest.java
index 540229bd65597..7716583e1a335 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/ConfigUtilsTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/ConfigUtilsTest.java
@@ -27,6 +27,7 @@
import java.util.Map;
import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertFalse;
import static org.junit.jupiter.api.Assertions.assertNotNull;
import static org.junit.jupiter.api.Assertions.assertNull;
@@ -161,10 +162,10 @@ public void testConfigMapToRedactedStringForEmptyMap() {
public void testConfigMapToRedactedStringWithSecrets() {
Map testMap1 = new HashMap<>();
testMap1.put("myString", "whatever");
- testMap1.put("myInt", Integer.valueOf(123));
+ testMap1.put("myInt", 123);
testMap1.put("myPassword", "foosecret");
testMap1.put("myString2", null);
- testMap1.put("myUnknown", Integer.valueOf(456));
+ testMap1.put("myUnknown", 456);
assertEquals("{myInt=123, myPassword=(redacted), myString=\"whatever\", myString2=null, myUnknown=(redacted)}",
ConfigUtils.configMapToRedactedString(testMap1, CONFIG));
}
@@ -172,7 +173,7 @@ public void testConfigMapToRedactedStringWithSecrets() {
@Test
public void testGetBoolean() {
String key = "test.key";
- Boolean defaultValue = true;
+ boolean defaultValue = true;
Map config = new HashMap<>();
config.put("some.other.key", false);
@@ -180,15 +181,15 @@ public void testGetBoolean() {
config = new HashMap<>();
config.put(key, false);
- assertEquals(false, ConfigUtils.getBoolean(config, key, defaultValue));
+ assertFalse(ConfigUtils.getBoolean(config, key, defaultValue));
config = new HashMap<>();
config.put(key, "false");
- assertEquals(false, ConfigUtils.getBoolean(config, key, defaultValue));
+ assertFalse(ConfigUtils.getBoolean(config, key, defaultValue));
config = new HashMap<>();
config.put(key, "not-a-boolean");
- assertEquals(false, ConfigUtils.getBoolean(config, key, defaultValue));
+ assertFalse(ConfigUtils.getBoolean(config, key, defaultValue));
config = new HashMap<>();
config.put(key, 5);
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/Crc32CTest.java b/clients/src/test/java/org/apache/kafka/common/utils/Crc32CTest.java
index c05df71372fdb..2c6d148e3a1e2 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/Crc32CTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/Crc32CTest.java
@@ -34,8 +34,8 @@ public void testUpdate() {
Checksum crc3 = Crc32C.create();
crc1.update(bytes, 0, len);
- for (int i = 0; i < len; i++)
- crc2.update(bytes[i]);
+ for (byte b : bytes)
+ crc2.update(b);
crc3.update(bytes, 0, len / 2);
crc3.update(bytes, len / 2, len - len / 2);
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/ImplicitLinkedHashMultiCollectionTest.java b/clients/src/test/java/org/apache/kafka/common/utils/ImplicitLinkedHashMultiCollectionTest.java
index 2d3c4b519a1f1..da75d40f30ca5 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/ImplicitLinkedHashMultiCollectionTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/ImplicitLinkedHashMultiCollectionTest.java
@@ -114,11 +114,9 @@ public void testEnlargement() {
new TestElement(101),
new TestElement(105)
};
- for (int i = 0; i < testElements.length; i++) {
- assertTrue(multiSet.add(testElements[i]));
- }
- for (int i = 0; i < testElements.length; i++) {
- assertFalse(multiSet.add(testElements[i]));
+ for (TestElement testElement : testElements) {
+ assertTrue(multiSet.add(testElement));
+ assertFalse(multiSet.add(testElement));
}
assertEquals(23, multiSet.numSlots());
assertEquals(testElements.length, multiSet.size());
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/MockScheduler.java b/clients/src/test/java/org/apache/kafka/common/utils/MockScheduler.java
index 98023f8a4e074..97560c3983d3f 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/MockScheduler.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/MockScheduler.java
@@ -74,11 +74,7 @@ public synchronized void addWaiter(long delayMs, KafkaFutureImpl waiter) {
waiter.complete(timeMs);
} else {
long triggerTimeMs = timeMs + delayMs;
- List> futures = waiters.get(triggerTimeMs);
- if (futures == null) {
- futures = new ArrayList<>();
- waiters.put(triggerTimeMs, futures);
- }
+ List> futures = waiters.computeIfAbsent(triggerTimeMs, k -> new ArrayList<>());
futures.add(waiter);
}
}
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/SanitizerTest.java b/clients/src/test/java/org/apache/kafka/common/utils/SanitizerTest.java
index 3024bd3583025..e78263a32e7e4 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/SanitizerTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/SanitizerTest.java
@@ -74,7 +74,7 @@ public interface TestStatMBean {
int getValue();
}
- public class TestStat implements TestStatMBean {
+ public static class TestStat implements TestStatMBean {
public int getValue() {
return 1;
}
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/SecurityUtilsTest.java b/clients/src/test/java/org/apache/kafka/common/utils/SecurityUtilsTest.java
index e651092bc44ed..a26fa4d6e8871 100644
--- a/clients/src/test/java/org/apache/kafka/common/utils/SecurityUtilsTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/SecurityUtilsTest.java
@@ -34,11 +34,11 @@
public class SecurityUtilsTest {
- private SecurityProviderCreator testScramSaslServerProviderCreator = new TestScramSaslServerProviderCreator();
- private SecurityProviderCreator testPlainSaslServerProviderCreator = new TestPlainSaslServerProviderCreator();
+ private final SecurityProviderCreator testScramSaslServerProviderCreator = new TestScramSaslServerProviderCreator();
+ private final SecurityProviderCreator testPlainSaslServerProviderCreator = new TestPlainSaslServerProviderCreator();
- private Provider testScramSaslServerProvider = testScramSaslServerProviderCreator.getProvider();
- private Provider testPlainSaslServerProvider = testPlainSaslServerProviderCreator.getProvider();
+ private final Provider testScramSaslServerProvider = testScramSaslServerProviderCreator.getProvider();
+ private final Provider testPlainSaslServerProvider = testPlainSaslServerProviderCreator.getProvider();
private void clearTestProviders() {
Security.removeProvider(testScramSaslServerProvider.getName());
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/UtilsTest.java b/clients/src/test/java/org/apache/kafka/common/utils/UtilsTest.java
index 467dd53ac76d0..89f050f68ca50 100755
--- a/clients/src/test/java/org/apache/kafka/common/utils/UtilsTest.java
+++ b/clients/src/test/java/org/apache/kafka/common/utils/UtilsTest.java
@@ -1008,7 +1008,7 @@ else if (failure == null)
};
}
- private class TestException extends Exception {
+ private static class TestException extends Exception {
final String key;
TestException(String key) {
this.key = key;
diff --git a/clients/src/test/java/org/apache/kafka/test/MetricsBench.java b/clients/src/test/java/org/apache/kafka/test/MetricsBench.java
index 4063d161f96a1..befd478da2466 100644
--- a/clients/src/test/java/org/apache/kafka/test/MetricsBench.java
+++ b/clients/src/test/java/org/apache/kafka/test/MetricsBench.java
@@ -50,7 +50,7 @@ public static void main(String[] args) {
for (int i = 0; i < iters; i++)
parent.record(i);
double elapsed = (System.nanoTime() - start) / (double) iters;
- System.out.println(String.format("%.2f ns per metric recording.", elapsed));
+ System.out.printf("%.2f ns per metric recording.%n", elapsed);
} finally {
metrics.close();
}
diff --git a/clients/src/test/java/org/apache/kafka/test/Microbenchmarks.java b/clients/src/test/java/org/apache/kafka/test/Microbenchmarks.java
index 4966694461226..ce31ef10e3039 100644
--- a/clients/src/test/java/org/apache/kafka/test/Microbenchmarks.java
+++ b/clients/src/test/java/org/apache/kafka/test/Microbenchmarks.java
@@ -78,34 +78,30 @@ public static void main(String[] args) throws Exception {
final Time time = Time.SYSTEM;
final AtomicBoolean done = new AtomicBoolean(false);
final Object lock = new Object();
- Thread t1 = new Thread() {
- public void run() {
- time.sleep(1);
- int counter = 0;
- long start = time.nanoseconds();
- for (int i = 0; i < iters; i++) {
- synchronized (lock) {
- counter++;
- }
+ Thread t1 = new Thread(() -> {
+ time.sleep(1);
+ int counter = 0;
+ long start1 = time.nanoseconds();
+ for (int i = 0; i < iters; i++) {
+ synchronized (lock) {
+ counter++;
}
- System.out.println("synchronized: " + ((time.nanoseconds() - start) / iters));
- System.out.println(counter);
- done.set(true);
}
- };
-
- Thread t2 = new Thread() {
- public void run() {
- int counter = 0;
- while (!done.get()) {
- time.sleep(1);
- synchronized (lock) {
- counter += 1;
- }
+ System.out.println("synchronized: " + ((time.nanoseconds() - start1) / iters));
+ System.out.println(counter);
+ done.set(true);
+ });
+
+ Thread t2 = new Thread(() -> {
+ int counter = 0;
+ while (!done.get()) {
+ time.sleep(1);
+ synchronized (lock) {
+ counter += 1;
}
- System.out.println("Counter: " + counter);
}
- };
+ System.out.println("Counter: " + counter);
+ });
t1.start();
t2.start();
@@ -115,34 +111,30 @@ public void run() {
System.out.println("Testing locks");
done.set(false);
final ReentrantLock lock2 = new ReentrantLock();
- Thread t3 = new Thread() {
- public void run() {
- time.sleep(1);
- int counter = 0;
- long start = time.nanoseconds();
- for (int i = 0; i < iters; i++) {
- lock2.lock();
- counter++;
- lock2.unlock();
- }
- System.out.println("lock: " + ((time.nanoseconds() - start) / iters));
- System.out.println(counter);
- done.set(true);
+ Thread t3 = new Thread(() -> {
+ time.sleep(1);
+ int counter = 0;
+ long start12 = time.nanoseconds();
+ for (int i = 0; i < iters; i++) {
+ lock2.lock();
+ counter++;
+ lock2.unlock();
}
- };
-
- Thread t4 = new Thread() {
- public void run() {
- int counter = 0;
- while (!done.get()) {
- time.sleep(1);
- lock2.lock();
- counter++;
- lock2.unlock();
- }
- System.out.println("Counter: " + counter);
+ System.out.println("lock: " + ((time.nanoseconds() - start12) / iters));
+ System.out.println(counter);
+ done.set(true);
+ });
+
+ Thread t4 = new Thread(() -> {
+ int counter = 0;
+ while (!done.get()) {
+ time.sleep(1);
+ lock2.lock();
+ counter++;
+ lock2.unlock();
}
- };
+ System.out.println("Counter: " + counter);
+ });
t3.start();
t4.start();
@@ -164,14 +156,12 @@ private static void benchMap(int numThreads, final int iters, final Map keys = new ArrayList<>(map.keySet());
final List threads = new ArrayList<>();
for (int i = 0; i < numThreads; i++) {
- threads.add(new Thread() {
- public void run() {
- long start = System.nanoTime();
- for (int j = 0; j < iters; j++)
- map.get(keys.get(j % threads.size()));
- System.out.println("Map access time: " + ((System.nanoTime() - start) / (double) iters));
- }
- });
+ threads.add(new Thread(() -> {
+ long start = System.nanoTime();
+ for (int j = 0; j < iters; j++)
+ map.get(keys.get(j % threads.size()));
+ System.out.println("Map access time: " + ((System.nanoTime() - start) / (double) iters));
+ }));
}
for (Thread thread : threads)
thread.start();