mirror of https://github.com/apache/kafka.git
KAFKA-10787 Apply spotless to `group-coordinator` and `group-coordinator-api` (#16298)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This commit is contained in:
parent
a5c71bd6cb
commit
6d9ef0e12a
|
@ -204,8 +204,6 @@ def determineCommitId() {
|
|||
def excludedSpotlessModules = [':clients',
|
||||
':connect:runtime',
|
||||
':core',
|
||||
':group-coordinator:group-coordinator-api', // https://github.com/apache/kafka/pull/16198
|
||||
':group-coordinator',
|
||||
':metadata',
|
||||
':raft',
|
||||
':server',
|
||||
|
@ -817,6 +815,7 @@ subprojects {
|
|||
apply plugin: 'com.diffplug.spotless'
|
||||
spotless {
|
||||
java {
|
||||
targetExclude 'src/generated/**/*.java'
|
||||
importOrder('kafka', 'org.apache.kafka', 'com', 'net', 'org', 'java', 'javax', '', '\\#')
|
||||
removeUnusedImports()
|
||||
}
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.coordinator.group;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.record.RecordBatch;
|
||||
import org.apache.kafka.common.requests.OffsetCommitRequest;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
import org.apache.kafka.coordinator.group.consumer.TopicMetadata;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupCurrentMemberAssignmentKey;
|
||||
|
@ -37,7 +38,6 @@ import org.apache.kafka.coordinator.group.generated.GroupMetadataKey;
|
|||
import org.apache.kafka.coordinator.group.generated.GroupMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitKey;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
import org.apache.kafka.server.common.MetadataVersion;
|
||||
|
||||
|
|
|
@ -46,9 +46,9 @@ import org.apache.kafka.common.message.TxnOffsetCommitRequestData;
|
|||
import org.apache.kafka.common.message.TxnOffsetCommitResponseData;
|
||||
import org.apache.kafka.common.protocol.Errors;
|
||||
import org.apache.kafka.common.requests.ApiError;
|
||||
import org.apache.kafka.common.requests.DescribeGroupsRequest;
|
||||
import org.apache.kafka.common.requests.DeleteGroupsRequest;
|
||||
import org.apache.kafka.common.requests.ConsumerGroupDescribeRequest;
|
||||
import org.apache.kafka.common.requests.DeleteGroupsRequest;
|
||||
import org.apache.kafka.common.requests.DescribeGroupsRequest;
|
||||
import org.apache.kafka.common.requests.OffsetCommitRequest;
|
||||
import org.apache.kafka.common.requests.RequestContext;
|
||||
import org.apache.kafka.common.requests.TransactionResult;
|
||||
|
@ -59,11 +59,11 @@ import org.apache.kafka.common.utils.Time;
|
|||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.coordinator.group.metrics.CoordinatorRuntimeMetrics;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetrics;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorShardBuilderSupplier;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorEventProcessor;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorLoader;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorRuntime;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorShardBuilderSupplier;
|
||||
import org.apache.kafka.coordinator.group.runtime.MultiThreadedEventProcessor;
|
||||
import org.apache.kafka.coordinator.group.runtime.PartitionWriter;
|
||||
import org.apache.kafka.image.MetadataDelta;
|
||||
|
@ -71,6 +71,7 @@ import org.apache.kafka.image.MetadataImage;
|
|||
import org.apache.kafka.server.record.BrokerCompressionType;
|
||||
import org.apache.kafka.server.util.FutureUtils;
|
||||
import org.apache.kafka.server.util.timer.Timer;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.time.Duration;
|
||||
|
|
|
@ -16,12 +16,13 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group;
|
||||
|
||||
import org.apache.kafka.common.message.ConsumerGroupDescribeResponseData;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.errors.ApiException;
|
||||
import org.apache.kafka.common.message.ConsumerGroupDescribeResponseData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.DescribeGroupsResponseData;
|
||||
import org.apache.kafka.common.message.DeleteGroupsResponseData;
|
||||
import org.apache.kafka.common.message.DescribeGroupsResponseData;
|
||||
import org.apache.kafka.common.message.HeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.HeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData;
|
||||
|
@ -37,7 +38,6 @@ import org.apache.kafka.common.message.OffsetFetchRequestData;
|
|||
import org.apache.kafka.common.message.OffsetFetchResponseData;
|
||||
import org.apache.kafka.common.message.SyncGroupRequestData;
|
||||
import org.apache.kafka.common.message.SyncGroupResponseData;
|
||||
import org.apache.kafka.common.errors.ApiException;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitRequestData;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitResponseData;
|
||||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
|
@ -66,14 +66,15 @@ import org.apache.kafka.coordinator.group.metrics.CoordinatorMetrics;
|
|||
import org.apache.kafka.coordinator.group.metrics.CoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetrics;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorShard;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorShardBuilder;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorTimer;
|
||||
import org.apache.kafka.image.MetadataDelta;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
|
|
@ -23,6 +23,7 @@ import org.apache.kafka.common.TopicPartition;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.errors.ApiException;
|
||||
import org.apache.kafka.common.errors.CoordinatorNotAvailableException;
|
||||
import org.apache.kafka.common.errors.FencedInstanceIdException;
|
||||
import org.apache.kafka.common.errors.FencedMemberEpochException;
|
||||
import org.apache.kafka.common.errors.GroupIdNotFoundException;
|
||||
import org.apache.kafka.common.errors.GroupMaxSizeReachedException;
|
||||
|
@ -32,18 +33,17 @@ import org.apache.kafka.common.errors.InvalidRequestException;
|
|||
import org.apache.kafka.common.errors.RebalanceInProgressException;
|
||||
import org.apache.kafka.common.errors.UnknownMemberIdException;
|
||||
import org.apache.kafka.common.errors.UnknownServerException;
|
||||
import org.apache.kafka.common.errors.UnsupportedAssignorException;
|
||||
import org.apache.kafka.common.errors.FencedInstanceIdException;
|
||||
import org.apache.kafka.common.errors.UnreleasedInstanceIdException;
|
||||
import org.apache.kafka.common.errors.UnsupportedAssignorException;
|
||||
import org.apache.kafka.common.message.ConsumerGroupDescribeResponseData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.DescribeGroupsResponseData;
|
||||
import org.apache.kafka.common.message.HeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.HeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocol;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocolCollection;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData;
|
||||
import org.apache.kafka.common.message.JoinGroupResponseData;
|
||||
import org.apache.kafka.common.message.LeaveGroupRequestData;
|
||||
import org.apache.kafka.common.message.LeaveGroupRequestData.MemberIdentity;
|
||||
|
@ -62,6 +62,9 @@ import org.apache.kafka.coordinator.group.api.assignor.ConsumerGroupPartitionAss
|
|||
import org.apache.kafka.coordinator.group.api.assignor.MemberAssignment;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignorException;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.Assignment;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
|
@ -83,9 +86,6 @@ import org.apache.kafka.coordinator.group.generated.ConsumerGroupTargetAssignmen
|
|||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupTargetAssignmentMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.generated.GroupMetadataKey;
|
||||
import org.apache.kafka.coordinator.group.generated.GroupMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorTimer;
|
||||
|
@ -96,6 +96,7 @@ import org.apache.kafka.image.TopicsImage;
|
|||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
import org.apache.kafka.timeline.TimelineHashMap;
|
||||
import org.apache.kafka.timeline.TimelineHashSet;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
@ -122,8 +123,6 @@ import static org.apache.kafka.common.protocol.Errors.UNKNOWN_SERVER_ERROR;
|
|||
import static org.apache.kafka.common.requests.ConsumerGroupHeartbeatRequest.LEAVE_GROUP_MEMBER_EPOCH;
|
||||
import static org.apache.kafka.common.requests.ConsumerGroupHeartbeatRequest.LEAVE_GROUP_STATIC_MEMBER_EPOCH;
|
||||
import static org.apache.kafka.common.requests.JoinGroupRequest.UNKNOWN_MEMBER_ID;
|
||||
import static org.apache.kafka.coordinator.group.Group.GroupType.CONSUMER;
|
||||
import static org.apache.kafka.coordinator.group.Group.GroupType.CLASSIC;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newCurrentAssignmentRecord;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newCurrentAssignmentTombstoneRecord;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newGroupEpochRecord;
|
||||
|
@ -131,6 +130,8 @@ import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newGro
|
|||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newMemberSubscriptionRecord;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newMemberSubscriptionTombstoneRecord;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpers.newTargetAssignmentTombstoneRecord;
|
||||
import static org.apache.kafka.coordinator.group.Group.GroupType.CLASSIC;
|
||||
import static org.apache.kafka.coordinator.group.Group.GroupType.CONSUMER;
|
||||
import static org.apache.kafka.coordinator.group.Utils.assignmentToString;
|
||||
import static org.apache.kafka.coordinator.group.Utils.ofSentinel;
|
||||
import static org.apache.kafka.coordinator.group.classic.ClassicGroupMember.EMPTY_ASSIGNMENT;
|
||||
|
|
|
@ -22,16 +22,16 @@ import org.apache.kafka.common.errors.GroupIdNotFoundException;
|
|||
import org.apache.kafka.common.errors.StaleMemberEpochException;
|
||||
import org.apache.kafka.common.message.OffsetCommitRequestData;
|
||||
import org.apache.kafka.common.message.OffsetCommitResponseData;
|
||||
import org.apache.kafka.common.message.OffsetCommitResponseData.OffsetCommitResponseTopic;
|
||||
import org.apache.kafka.common.message.OffsetCommitResponseData.OffsetCommitResponsePartition;
|
||||
import org.apache.kafka.common.message.OffsetCommitResponseData.OffsetCommitResponseTopic;
|
||||
import org.apache.kafka.common.message.OffsetDeleteRequestData;
|
||||
import org.apache.kafka.common.message.OffsetDeleteResponseData;
|
||||
import org.apache.kafka.common.message.OffsetFetchRequestData;
|
||||
import org.apache.kafka.common.message.OffsetFetchResponseData;
|
||||
import org.apache.kafka.common.message.OffsetDeleteResponseData;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitRequestData;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitResponseData;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitResponseData.TxnOffsetCommitResponseTopic;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitResponseData.TxnOffsetCommitResponsePartition;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitResponseData.TxnOffsetCommitResponseTopic;
|
||||
import org.apache.kafka.common.protocol.Errors;
|
||||
import org.apache.kafka.common.record.RecordBatch;
|
||||
import org.apache.kafka.common.requests.OffsetCommitRequest;
|
||||
|
@ -39,18 +39,19 @@ import org.apache.kafka.common.requests.RequestContext;
|
|||
import org.apache.kafka.common.requests.TransactionResult;
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitKey;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitKey;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetrics;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.image.MetadataDelta;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
import org.apache.kafka.timeline.TimelineHashMap;
|
||||
import org.apache.kafka.timeline.TimelineHashSet;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
@ -62,8 +63,8 @@ import java.util.Map;
|
|||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.Set;
|
||||
import java.util.concurrent.atomic.AtomicInteger;
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
import java.util.concurrent.atomic.AtomicInteger;
|
||||
import java.util.function.Consumer;
|
||||
|
||||
import static org.apache.kafka.common.requests.OffsetFetchResponse.INVALID_OFFSET;
|
||||
|
|
|
@ -253,4 +253,3 @@ public class RangeAssignor implements ConsumerGroupPartitionAssignor {
|
|||
return new GroupAssignment(newTargetAssignment);
|
||||
}
|
||||
}
|
||||
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.kafka.coordinator.group.api.assignor.GroupAssignment;
|
|||
import org.apache.kafka.coordinator.group.api.assignor.GroupSpec;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignorException;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscribedTopicDescriber;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
|
|
@ -24,6 +24,7 @@ import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignorExceptio
|
|||
import org.apache.kafka.coordinator.group.api.assignor.SubscribedTopicDescriber;
|
||||
import org.apache.kafka.coordinator.group.consumer.MemberAssignmentImpl;
|
||||
import org.apache.kafka.server.common.TopicIdPartition;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
|
|
@ -28,25 +28,26 @@ import org.apache.kafka.common.errors.IllegalGenerationException;
|
|||
import org.apache.kafka.common.errors.UnknownMemberIdException;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocolCollection;
|
||||
import org.apache.kafka.common.message.JoinGroupResponseData;
|
||||
import org.apache.kafka.common.message.JoinGroupResponseData.JoinGroupResponseMember;
|
||||
import org.apache.kafka.common.message.ListGroupsResponseData;
|
||||
import org.apache.kafka.common.message.SyncGroupResponseData;
|
||||
import org.apache.kafka.common.message.JoinGroupResponseData.JoinGroupResponseMember;
|
||||
import org.apache.kafka.common.protocol.Errors;
|
||||
import org.apache.kafka.common.protocol.types.SchemaException;
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecord;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecordHelpers;
|
||||
import org.apache.kafka.coordinator.group.Group;
|
||||
import org.apache.kafka.coordinator.group.OffsetExpirationCondition;
|
||||
import org.apache.kafka.coordinator.group.OffsetExpirationConditionImpl;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecord;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecordHelpers;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.image.TopicImage;
|
||||
import org.apache.kafka.image.TopicsImage;
|
||||
import org.apache.kafka.server.common.MetadataVersion;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
|
|
@ -28,11 +28,11 @@ import org.apache.kafka.common.errors.UnsupportedVersionException;
|
|||
import org.apache.kafka.common.message.ConsumerGroupDescribeResponseData;
|
||||
import org.apache.kafka.common.message.ListGroupsResponseData;
|
||||
import org.apache.kafka.common.protocol.Errors;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecord;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecordHelpers;
|
||||
import org.apache.kafka.coordinator.group.Group;
|
||||
import org.apache.kafka.coordinator.group.OffsetExpirationCondition;
|
||||
import org.apache.kafka.coordinator.group.OffsetExpirationConditionImpl;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecord;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecordHelpers;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupMemberMetadataValue;
|
||||
|
|
|
@ -18,11 +18,11 @@ package org.apache.kafka.coordinator.group.consumer;
|
|||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecord;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.GroupAssignment;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.MemberAssignment;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignor;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignorException;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
||||
import org.apache.kafka.image.TopicsImage;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
|
|
@ -16,12 +16,13 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group.metrics;
|
||||
|
||||
import com.yammer.metrics.core.MetricName;
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import com.yammer.metrics.core.MetricName;
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
|
||||
/**
|
||||
* CoordinatorMetrics contain all coordinator related metrics. It delegates metrics collection to
|
||||
* {@link CoordinatorMetricsShard}s and aggregates them all when it reports to the metrics registry.
|
||||
|
|
|
@ -16,7 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group.metrics;
|
||||
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
import org.apache.kafka.common.MetricName;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.metrics.Gauge;
|
||||
|
@ -25,11 +24,13 @@ import org.apache.kafka.common.metrics.Sensor;
|
|||
import org.apache.kafka.common.metrics.stats.Meter;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.coordinator.group.Group;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup.ConsumerGroupState;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup.ConsumerGroupState;
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
import java.util.Map;
|
||||
|
|
|
@ -19,8 +19,8 @@ package org.apache.kafka.coordinator.group.metrics;
|
|||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.metrics.Sensor;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup.ConsumerGroupState;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup.ConsumerGroupState;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
import org.apache.kafka.timeline.TimelineLong;
|
||||
|
||||
|
|
|
@ -37,8 +37,8 @@ import org.apache.kafka.common.utils.BufferSupplier;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.coordinator.group.metrics.CoordinatorRuntimeMetrics;
|
||||
import org.apache.kafka.coordinator.group.metrics.CoordinatorMetrics;
|
||||
import org.apache.kafka.coordinator.group.metrics.CoordinatorRuntimeMetrics;
|
||||
import org.apache.kafka.deferred.DeferredEvent;
|
||||
import org.apache.kafka.deferred.DeferredEventQueue;
|
||||
import org.apache.kafka.image.MetadataDelta;
|
||||
|
@ -48,6 +48,7 @@ import org.apache.kafka.server.util.timer.TimerTask;
|
|||
import org.apache.kafka.storage.internals.log.LogConfig;
|
||||
import org.apache.kafka.storage.internals.log.VerificationGuard;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
|
|
@ -20,6 +20,7 @@ import org.apache.kafka.common.TopicPartition;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.metrics.CoordinatorRuntimeMetrics;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.List;
|
||||
|
|
|
@ -22,6 +22,7 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.image.MetadataDelta;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
/**
|
||||
|
|
|
@ -29,6 +29,7 @@ import org.apache.kafka.coordinator.group.generated.ConsumerGroupPartitionMetada
|
|||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupTargetAssignmentMemberValue;
|
||||
import org.apache.kafka.coordinator.group.generated.GroupMetadataValue;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
|
||||
import org.opentest4j.AssertionFailedError;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
|
|
@ -22,6 +22,9 @@ import org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProt
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
import org.apache.kafka.coordinator.group.consumer.MemberState;
|
||||
import org.apache.kafka.coordinator.group.consumer.TopicMetadata;
|
||||
|
@ -41,12 +44,10 @@ import org.apache.kafka.coordinator.group.generated.GroupMetadataKey;
|
|||
import org.apache.kafka.coordinator.group.generated.GroupMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitKey;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
import org.apache.kafka.server.common.MetadataVersion;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.Arguments;
|
||||
|
@ -58,7 +59,6 @@ import java.util.Arrays;
|
|||
import java.util.Collections;
|
||||
import java.util.HashMap;
|
||||
import java.util.HashSet;
|
||||
|
||||
import java.util.LinkedHashMap;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
|
|
|
@ -36,6 +36,7 @@ import org.apache.kafka.coordinator.group.generated.OffsetCommitKey;
|
|||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorLoader;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.coordinator.group;
|
|||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupMetadataKey;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupMetadataValue;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.coordinator.group;
|
|||
import org.apache.kafka.common.record.CompressionType;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.ConsumerGroupPartitionAssignor;
|
||||
import org.apache.kafka.coordinator.group.assignor.RangeAssignor;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -35,20 +35,20 @@ import org.apache.kafka.common.errors.UnknownTopicOrPartitionException;
|
|||
import org.apache.kafka.common.message.ConsumerGroupDescribeResponseData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.DescribeGroupsResponseData;
|
||||
import org.apache.kafka.common.message.DeleteGroupsResponseData;
|
||||
import org.apache.kafka.common.message.DescribeGroupsResponseData;
|
||||
import org.apache.kafka.common.message.HeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.HeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData;
|
||||
import org.apache.kafka.common.message.JoinGroupResponseData;
|
||||
import org.apache.kafka.common.message.LeaveGroupRequestData;
|
||||
import org.apache.kafka.common.message.LeaveGroupResponseData;
|
||||
import org.apache.kafka.common.message.ListGroupsRequestData;
|
||||
import org.apache.kafka.common.message.ListGroupsResponseData;
|
||||
import org.apache.kafka.common.message.OffsetDeleteRequestData;
|
||||
import org.apache.kafka.common.message.OffsetDeleteResponseData;
|
||||
import org.apache.kafka.common.message.OffsetFetchRequestData;
|
||||
import org.apache.kafka.common.message.OffsetFetchResponseData;
|
||||
import org.apache.kafka.common.message.LeaveGroupRequestData;
|
||||
import org.apache.kafka.common.message.LeaveGroupResponseData;
|
||||
import org.apache.kafka.common.message.SyncGroupRequestData;
|
||||
import org.apache.kafka.common.message.SyncGroupResponseData;
|
||||
import org.apache.kafka.common.message.TxnOffsetCommitRequestData;
|
||||
|
@ -71,6 +71,7 @@ import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetrics;
|
|||
import org.apache.kafka.coordinator.group.runtime.CoordinatorRuntime;
|
||||
import org.apache.kafka.server.record.BrokerCompressionType;
|
||||
import org.apache.kafka.server.util.FutureUtils;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.Arguments;
|
||||
|
|
|
@ -53,6 +53,7 @@ import org.apache.kafka.coordinator.group.metrics.CoordinatorMetricsShard;
|
|||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.EnumSource;
|
||||
|
|
|
@ -33,9 +33,9 @@ import org.apache.kafka.common.errors.RebalanceInProgressException;
|
|||
import org.apache.kafka.common.errors.UnknownMemberIdException;
|
||||
import org.apache.kafka.common.errors.UnknownServerException;
|
||||
import org.apache.kafka.common.errors.UnknownTopicOrPartitionException;
|
||||
import org.apache.kafka.common.errors.UnreleasedInstanceIdException;
|
||||
import org.apache.kafka.common.errors.UnsupportedAssignorException;
|
||||
import org.apache.kafka.common.message.ConsumerGroupDescribeResponseData;
|
||||
import org.apache.kafka.common.errors.UnreleasedInstanceIdException;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatRequestData;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData;
|
||||
import org.apache.kafka.common.message.ConsumerProtocolAssignment;
|
||||
|
@ -64,25 +64,26 @@ import org.apache.kafka.coordinator.group.MockCoordinatorTimer.ScheduledTimeout;
|
|||
import org.apache.kafka.coordinator.group.api.assignor.ConsumerGroupPartitionAssignor;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.GroupAssignment;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.MemberAssignment;
|
||||
import org.apache.kafka.coordinator.group.consumer.MemberAssignmentImpl;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignorException;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.Assignment;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupBuilder;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
import org.apache.kafka.coordinator.group.consumer.MemberAssignmentImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.MemberState;
|
||||
import org.apache.kafka.coordinator.group.consumer.TopicMetadata;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupMemberMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.generated.GroupMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.image.MetadataDelta;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.image.MetadataProvenance;
|
||||
import org.apache.kafka.server.common.MetadataVersion;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
@ -102,11 +103,11 @@ import java.util.stream.Collectors;
|
|||
import java.util.stream.IntStream;
|
||||
import java.util.stream.Stream;
|
||||
|
||||
import static org.apache.kafka.common.protocol.Errors.NOT_COORDINATOR;
|
||||
import static org.apache.kafka.common.requests.ConsumerGroupHeartbeatRequest.LEAVE_GROUP_STATIC_MEMBER_EPOCH;
|
||||
import static org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocol;
|
||||
import static org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocolCollection;
|
||||
import static org.apache.kafka.common.protocol.Errors.NOT_COORDINATOR;
|
||||
import static org.apache.kafka.common.requests.ConsumerGroupHeartbeatRequest.LEAVE_GROUP_MEMBER_EPOCH;
|
||||
import static org.apache.kafka.common.requests.ConsumerGroupHeartbeatRequest.LEAVE_GROUP_STATIC_MEMBER_EPOCH;
|
||||
import static org.apache.kafka.common.requests.JoinGroupRequest.UNKNOWN_MEMBER_ID;
|
||||
import static org.apache.kafka.common.utils.Utils.mkSet;
|
||||
import static org.apache.kafka.coordinator.group.Assertions.assertRecordEquals;
|
||||
|
@ -115,15 +116,15 @@ import static org.apache.kafka.coordinator.group.Assertions.assertResponseEquals
|
|||
import static org.apache.kafka.coordinator.group.Assertions.assertUnorderedListEquals;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkTopicAssignment;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpersTest.mkMapOfPartitionRacks;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.EMPTY_RESULT;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.appendGroupMetadataErrorToResponseError;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.classicGroupHeartbeatKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.classicGroupJoinKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.classicGroupSyncKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.consumerGroupJoinKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.consumerGroupRebalanceTimeoutKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.consumerGroupSessionTimeoutKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.EMPTY_RESULT;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.classicGroupHeartbeatKey;
|
||||
import static org.apache.kafka.coordinator.group.GroupMetadataManager.classicGroupSyncKey;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpersTest.mkMapOfPartitionRacks;
|
||||
import static org.apache.kafka.coordinator.group.classic.ClassicGroupMember.EMPTY_ASSIGNMENT;
|
||||
import static org.apache.kafka.coordinator.group.classic.ClassicGroupState.COMPLETING_REBALANCE;
|
||||
import static org.apache.kafka.coordinator.group.classic.ClassicGroupState.DEAD;
|
||||
|
|
|
@ -20,6 +20,7 @@ import org.apache.kafka.common.message.OffsetCommitRequestData;
|
|||
import org.apache.kafka.common.message.TxnOffsetCommitRequestData;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.server.util.MockTime;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.OptionalInt;
|
||||
|
|
|
@ -48,20 +48,21 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.annotation.ApiKeyVersionsSource;
|
||||
import org.apache.kafka.coordinator.group.assignor.RangeAssignor;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupMemberMetadataValue;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitKey;
|
||||
import org.apache.kafka.coordinator.group.generated.OffsetCommitValue;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorResult;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
import org.apache.kafka.server.common.MetadataVersion;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.EnumSource;
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
|||
import org.apache.kafka.coordinator.group.consumer.Assignment;
|
||||
import org.apache.kafka.coordinator.group.consumer.GroupSpecImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.MemberSubscriptionAndAssignmentImpl;
|
||||
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
|
|
|
@ -25,6 +25,7 @@ import org.apache.kafka.coordinator.group.consumer.GroupSpecImpl;
|
|||
import org.apache.kafka.coordinator.group.consumer.MemberSubscriptionAndAssignmentImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.SubscribedTopicDescriberImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.TopicMetadata;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
@ -39,10 +40,10 @@ import java.util.TreeMap;
|
|||
|
||||
import static org.apache.kafka.common.utils.Utils.mkSet;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.assertAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.invertedTargetAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkOrderedAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkTopicAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.invertedTargetAssignment;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpersTest.mkMapOfPartitionRacks;
|
||||
import static org.apache.kafka.coordinator.group.api.assignor.SubscriptionType.HOMOGENEOUS;
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
|
|
@ -25,6 +25,7 @@ import org.apache.kafka.coordinator.group.consumer.GroupSpecImpl;
|
|||
import org.apache.kafka.coordinator.group.consumer.MemberSubscriptionAndAssignmentImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.SubscribedTopicDescriberImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.TopicMetadata;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
@ -35,9 +36,9 @@ import java.util.Set;
|
|||
import java.util.TreeMap;
|
||||
|
||||
import static org.apache.kafka.common.utils.Utils.mkSet;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.invertedTargetAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkTopicAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.invertedTargetAssignment;
|
||||
import static org.apache.kafka.coordinator.group.api.assignor.SubscriptionType.HETEROGENEOUS;
|
||||
import static org.apache.kafka.coordinator.group.api.assignor.SubscriptionType.HOMOGENEOUS;
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
|
|
@ -25,6 +25,7 @@ import org.apache.kafka.coordinator.group.consumer.GroupSpecImpl;
|
|||
import org.apache.kafka.coordinator.group.consumer.MemberSubscriptionAndAssignmentImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.SubscribedTopicDescriberImpl;
|
||||
import org.apache.kafka.coordinator.group.consumer.TopicMetadata;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
@ -36,9 +37,9 @@ import java.util.TreeMap;
|
|||
|
||||
import static org.apache.kafka.common.utils.Utils.mkSet;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.assertAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.invertedTargetAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.mkTopicAssignment;
|
||||
import static org.apache.kafka.coordinator.group.AssignmentTestUtil.invertedTargetAssignment;
|
||||
import static org.apache.kafka.coordinator.group.CoordinatorRecordHelpersTest.mkMapOfPartitionRacks;
|
||||
import static org.apache.kafka.coordinator.group.api.assignor.SubscriptionType.HETEROGENEOUS;
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
|
|
@ -20,6 +20,7 @@ package org.apache.kafka.coordinator.group.classic;
|
|||
import org.apache.kafka.common.message.DescribeGroupsResponseData;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocol;
|
||||
import org.apache.kafka.common.message.JoinGroupRequestData.JoinGroupRequestProtocolCollection;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -41,6 +41,7 @@ import org.apache.kafka.coordinator.group.OffsetExpirationCondition;
|
|||
import org.apache.kafka.coordinator.group.OffsetExpirationConditionImpl;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -18,6 +18,7 @@ package org.apache.kafka.coordinator.group.consumer;
|
|||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupTargetAssignmentMemberValue;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
|
|
@ -23,6 +23,7 @@ import org.apache.kafka.coordinator.group.MetadataImageBuilder;
|
|||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupCurrentMemberAssignmentValue;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupMemberMetadataValue;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
|
|
@ -38,6 +38,7 @@ import org.apache.kafka.coordinator.group.generated.ConsumerGroupMemberMetadataV
|
|||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorMetricsShard;
|
||||
import org.apache.kafka.image.MetadataImage;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
||||
|
@ -60,9 +61,9 @@ import static org.apache.kafka.coordinator.group.api.assignor.SubscriptionType.H
|
|||
import static org.junit.jupiter.api.Assertions.assertDoesNotThrow;
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertNull;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
import static org.junit.jupiter.api.Assertions.assertTrue;
|
||||
import static org.junit.jupiter.api.Assertions.assertNull;
|
||||
import static org.mockito.Mockito.mock;
|
||||
import static org.mockito.Mockito.times;
|
||||
import static org.mockito.Mockito.verify;
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.coordinator.group.consumer;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.errors.FencedMemberEpochException;
|
||||
import org.apache.kafka.common.message.ConsumerGroupHeartbeatRequestData;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Arrays;
|
||||
|
|
|
@ -17,6 +17,7 @@
|
|||
package org.apache.kafka.coordinator.group.consumer;
|
||||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
|
|
|
@ -19,12 +19,13 @@ package org.apache.kafka.coordinator.group.consumer;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.coordinator.group.AssignmentTestUtil;
|
||||
import org.apache.kafka.coordinator.group.MetadataImageBuilder;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.MemberSubscription;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.GroupAssignment;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.MemberAssignment;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.MemberSubscription;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.PartitionAssignor;
|
||||
import org.apache.kafka.coordinator.group.api.assignor.SubscriptionType;
|
||||
import org.apache.kafka.image.TopicsImage;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Arrays;
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.coordinator.group.consumer;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.coordinator.group.MetadataImageBuilder;
|
||||
import org.apache.kafka.image.TopicsImage;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -18,6 +18,7 @@ package org.apache.kafka.coordinator.group.consumer;
|
|||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.coordinator.group.generated.ConsumerGroupPartitionMetadataValue;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -16,17 +16,19 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group.metrics;
|
||||
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.metrics.Metrics;
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroup;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroupMember;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -16,16 +16,18 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group.metrics;
|
||||
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
import org.apache.kafka.common.MetricName;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.metrics.Metrics;
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup.ConsumerGroupState;
|
||||
import org.apache.kafka.coordinator.group.classic.ClassicGroupState;
|
||||
import org.apache.kafka.coordinator.group.consumer.ConsumerGroup.ConsumerGroupState;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Arrays;
|
||||
|
|
|
@ -22,6 +22,7 @@ import org.apache.kafka.common.metrics.Metrics;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.runtime.CoordinatorRuntime.CoordinatorState;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Arrays;
|
||||
|
|
|
@ -16,11 +16,12 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group.metrics;
|
||||
|
||||
import com.yammer.metrics.core.Gauge;
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
import org.apache.kafka.common.MetricName;
|
||||
import org.apache.kafka.common.metrics.Metrics;
|
||||
|
||||
import com.yammer.metrics.core.Gauge;
|
||||
import com.yammer.metrics.core.MetricsRegistry;
|
||||
|
||||
import java.util.Set;
|
||||
import java.util.TreeSet;
|
||||
|
||||
|
|
|
@ -17,6 +17,7 @@
|
|||
package org.apache.kafka.coordinator.group.runtime;
|
||||
|
||||
import org.apache.kafka.coordinator.group.CoordinatorRecord;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -50,6 +50,7 @@ import org.apache.kafka.storage.internals.log.VerificationGuard;
|
|||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
import org.apache.kafka.timeline.TimelineHashMap;
|
||||
import org.apache.kafka.timeline.TimelineHashSet;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.EnumSource;
|
||||
|
|
|
@ -16,6 +16,8 @@
|
|||
*/
|
||||
package org.apache.kafka.coordinator.group.runtime;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.ArrayList;
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
|
@ -28,8 +30,6 @@ import java.util.concurrent.TimeUnit;
|
|||
import java.util.concurrent.TimeoutException;
|
||||
import java.util.stream.Collectors;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertNotNull;
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.coordinator.group.metrics.GroupCoordinatorRuntimeMetrics;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.api.Timeout;
|
||||
import org.mockito.ArgumentCaptor;
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.coordinator.group.MockCoordinatorTimer;
|
||||
import org.apache.kafka.timeline.SnapshotRegistry;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.apache.kafka.coordinator.group.runtime.CoordinatorRuntimeTest.MockCoordinatorShard;
|
||||
|
|
Loading…
Reference in New Issue