mirror of https://github.com/apache/kafka.git
KAFKA-17238 Move VoterSet and ReplicaKey from raft.internals to raft (#16775)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This commit is contained in:
parent
7031855570
commit
81f0b13a70
|
@ -42,8 +42,7 @@ import org.apache.kafka.coordinator.group.{CoordinatorRecord, CoordinatorRecordS
|
|||
import org.apache.kafka.coordinator.group.generated.{ConsumerGroupMemberMetadataValue, ConsumerGroupMetadataKey, ConsumerGroupMetadataValue, GroupMetadataKey, GroupMetadataValue}
|
||||
import org.apache.kafka.coordinator.transaction.TransactionLogConfigs
|
||||
import org.apache.kafka.metadata.MetadataRecordSerde
|
||||
import org.apache.kafka.raft.{KafkaRaftClient, OffsetAndEpoch}
|
||||
import org.apache.kafka.raft.internals.VoterSetTest
|
||||
import org.apache.kafka.raft.{KafkaRaftClient, OffsetAndEpoch, VoterSetTest}
|
||||
import org.apache.kafka.server.common.{ApiMessageAndVersion, KRaftVersion}
|
||||
import org.apache.kafka.server.config.ServerLogConfigs
|
||||
import org.apache.kafka.server.log.remote.metadata.storage.serialization.RemoteLogMetadataSerde
|
||||
|
|
|
@ -27,7 +27,7 @@ import org.apache.kafka.metadata.properties.MetaPropertiesEnsemble;
|
|||
import org.apache.kafka.metadata.properties.MetaPropertiesVersion;
|
||||
import org.apache.kafka.raft.DynamicVoters;
|
||||
import org.apache.kafka.raft.KafkaRaftClient;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion;
|
||||
import org.apache.kafka.server.common.FeatureVersion;
|
||||
import org.apache.kafka.server.common.Features;
|
||||
|
|
|
@ -20,8 +20,6 @@ import org.apache.kafka.common.Uuid;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.Timer;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
|
|
|
@ -20,8 +20,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.feature.SupportedVersionRange;
|
||||
import org.apache.kafka.common.network.ListenerName;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import java.net.InetSocketAddress;
|
||||
import java.util.Collections;
|
||||
|
|
|
@ -17,7 +17,6 @@
|
|||
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collection;
|
||||
|
|
|
@ -17,7 +17,6 @@
|
|||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import java.util.List;
|
||||
import java.util.Objects;
|
||||
|
|
|
@ -16,8 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import java.io.Closeable;
|
||||
import java.util.Optional;
|
||||
|
||||
|
|
|
@ -21,7 +21,6 @@ import org.apache.kafka.common.network.ListenerName;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.Timer;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.snapshot.RawSnapshotWriter;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
|
|
@ -77,10 +77,8 @@ import org.apache.kafka.raft.internals.KafkaRaftMetrics;
|
|||
import org.apache.kafka.raft.internals.MemoryBatchReader;
|
||||
import org.apache.kafka.raft.internals.RecordsBatchReader;
|
||||
import org.apache.kafka.raft.internals.RemoveVoterHandler;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.ThresholdPurgatory;
|
||||
import org.apache.kafka.raft.internals.UpdateVoterHandler;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
import org.apache.kafka.snapshot.NotifyingRawSnapshotWriter;
|
||||
|
|
|
@ -30,8 +30,6 @@ import org.apache.kafka.common.utils.Timer;
|
|||
import org.apache.kafka.raft.internals.AddVoterHandlerState;
|
||||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.RemoveVoterHandlerState;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
|
|
@ -22,8 +22,6 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.KRaftControlRecordStateMachine;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
|
|
|
@ -42,7 +42,6 @@ import org.apache.kafka.common.network.ListenerName;
|
|||
import org.apache.kafka.common.protocol.ApiKeys;
|
||||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
import org.apache.kafka.common.protocol.Errors;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import java.net.InetSocketAddress;
|
||||
import java.util.Collection;
|
||||
|
|
|
@ -14,7 +14,7 @@
|
|||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
|
|
@ -19,7 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.Timer;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
|
|
|
@ -19,7 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.Timer;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
|
|
|
@ -14,7 +14,7 @@
|
|||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
|
@ -22,7 +22,6 @@ import org.apache.kafka.common.feature.SupportedVersionRange;
|
|||
import org.apache.kafka.common.message.VotersRecord;
|
||||
import org.apache.kafka.common.network.ListenerName;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.Endpoints;
|
||||
|
||||
import java.net.InetSocketAddress;
|
||||
import java.util.Collections;
|
||||
|
@ -49,7 +48,7 @@ import java.util.stream.Stream;
|
|||
public final class VoterSet {
|
||||
private final Map<Integer, VoterNode> voters;
|
||||
|
||||
VoterSet(Map<Integer, VoterNode> voters) {
|
||||
public VoterSet(Map<Integer, VoterNode> voters) {
|
||||
if (voters.isEmpty()) {
|
||||
throw new IllegalArgumentException("Voters cannot be empty");
|
||||
}
|
|
@ -29,6 +29,8 @@ import org.apache.kafka.raft.Endpoints;
|
|||
import org.apache.kafka.raft.LeaderState;
|
||||
import org.apache.kafka.raft.LogOffsetMetadata;
|
||||
import org.apache.kafka.raft.RaftUtil;
|
||||
import org.apache.kafka.raft.ReplicaKey;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
|
|
@ -20,6 +20,7 @@ package org.apache.kafka.raft.internals;
|
|||
import org.apache.kafka.common.message.AddRaftVoterResponseData;
|
||||
import org.apache.kafka.common.utils.Timer;
|
||||
import org.apache.kafka.raft.Endpoints;
|
||||
import org.apache.kafka.raft.ReplicaKey;
|
||||
|
||||
import java.util.OptionalLong;
|
||||
import java.util.concurrent.CompletableFuture;
|
||||
|
|
|
@ -25,6 +25,7 @@ import org.apache.kafka.raft.ControlRecord;
|
|||
import org.apache.kafka.raft.Isolation;
|
||||
import org.apache.kafka.raft.LogFetchInfo;
|
||||
import org.apache.kafka.raft.ReplicatedLog;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
|
|
|
@ -28,6 +28,7 @@ import org.apache.kafka.common.metrics.stats.WindowedSum;
|
|||
import org.apache.kafka.raft.LogOffsetMetadata;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.QuorumState;
|
||||
import org.apache.kafka.raft.ReplicaKey;
|
||||
|
||||
import java.util.Arrays;
|
||||
import java.util.OptionalLong;
|
||||
|
|
|
@ -24,6 +24,8 @@ import org.apache.kafka.common.utils.Time;
|
|||
import org.apache.kafka.raft.LeaderState;
|
||||
import org.apache.kafka.raft.LogOffsetMetadata;
|
||||
import org.apache.kafka.raft.RaftUtil;
|
||||
import org.apache.kafka.raft.ReplicaKey;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
|
|
@ -27,6 +27,8 @@ import org.apache.kafka.raft.LeaderAndEpoch;
|
|||
import org.apache.kafka.raft.LeaderState;
|
||||
import org.apache.kafka.raft.LogOffsetMetadata;
|
||||
import org.apache.kafka.raft.RaftUtil;
|
||||
import org.apache.kafka.raft.ReplicaKey;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
import java.util.Optional;
|
||||
|
|
|
@ -16,6 +16,8 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
|
||||
|
|
|
@ -28,9 +28,9 @@ import org.apache.kafka.common.record.RecordBatch;
|
|||
import org.apache.kafka.common.record.TimestampType;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.BatchAccumulator.CompletedBatch;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
|
||||
|
|
|
@ -20,9 +20,6 @@ import org.apache.kafka.common.Uuid;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -19,8 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.feature.SupportedVersionRange;
|
||||
import org.apache.kafka.common.network.ListenerName;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
|
|
|
@ -17,7 +17,6 @@
|
|||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
|
|
|
@ -19,7 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -20,7 +20,6 @@ import org.apache.kafka.common.Uuid;
|
|||
import org.apache.kafka.common.protocol.types.TaggedFields;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
|
|
|
@ -20,7 +20,6 @@ import org.apache.kafka.common.network.ListenerName;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -47,7 +47,6 @@ import org.apache.kafka.common.requests.VoteResponse;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.common.utils.annotation.ApiKeyVersionsSource;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
|
|
@ -36,9 +36,6 @@ import org.apache.kafka.common.record.RecordBatch;
|
|||
import org.apache.kafka.common.record.Records;
|
||||
import org.apache.kafka.common.record.TimestampType;
|
||||
import org.apache.kafka.common.utils.BufferSupplier;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
import org.apache.kafka.server.common.Features;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotReader;
|
||||
|
|
|
@ -26,10 +26,7 @@ import org.apache.kafka.common.record.MemoryRecords;
|
|||
import org.apache.kafka.common.record.UnalignedMemoryRecords;
|
||||
import org.apache.kafka.common.requests.FetchSnapshotRequest;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.StringSerde;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
import org.apache.kafka.snapshot.RawSnapshotWriter;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotWriter;
|
||||
|
|
|
@ -40,9 +40,6 @@ import org.apache.kafka.common.requests.FetchRequest;
|
|||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.errors.BufferAllocationException;
|
||||
import org.apache.kafka.raft.errors.NotLeaderException;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
|
|
@ -20,9 +20,6 @@ import org.apache.kafka.common.Uuid;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
|
|
@ -18,7 +18,6 @@ package org.apache.kafka.raft;
|
|||
|
||||
import org.apache.kafka.common.network.ListenerName;
|
||||
import org.apache.kafka.common.protocol.ApiKeys;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
|
||||
import java.util.ArrayList;
|
||||
import java.util.HashMap;
|
||||
|
|
|
@ -22,9 +22,6 @@ import org.apache.kafka.common.utils.MockTime;
|
|||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.KRaftControlRecordStateMachine;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
import org.apache.kafka.server.common.Features;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
|
|
|
@ -63,9 +63,7 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.BatchBuilder;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.StringSerde;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.server.common.Features;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
|
|
|
@ -19,8 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -19,7 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -19,7 +19,6 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -14,14 +14,13 @@
|
|||
* See the License for the specific language governing permissions and
|
||||
* limitations under the License.
|
||||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.feature.SupportedVersionRange;
|
||||
import org.apache.kafka.common.network.ListenerName;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.Endpoints;
|
||||
import org.apache.kafka.server.common.Features;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
|
@ -24,6 +24,8 @@ import org.apache.kafka.common.utils.BufferSupplier;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.raft.MockLog;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.raft.VoterSetTest;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotWriter;
|
||||
|
|
|
@ -26,6 +26,9 @@ import org.apache.kafka.raft.LogOffsetMetadata;
|
|||
import org.apache.kafka.raft.MockQuorumStateStore;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.QuorumState;
|
||||
import org.apache.kafka.raft.ReplicaKey;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.raft.VoterSetTest;
|
||||
import org.apache.kafka.server.common.Features;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
|
||||
|
|
|
@ -40,6 +40,8 @@ import org.apache.kafka.common.utils.MockTime;
|
|||
import org.apache.kafka.raft.Batch;
|
||||
import org.apache.kafka.raft.ControlRecord;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.raft.VoterSetTest;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
import org.apache.kafka.snapshot.MockRawSnapshotWriter;
|
||||
|
|
|
@ -16,6 +16,9 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.raft.VoterSetTest;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.HashMap;
|
||||
|
|
|
@ -25,9 +25,9 @@ import org.apache.kafka.common.utils.BufferSupplier;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.raft.Batch;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.VoterSet;
|
||||
import org.apache.kafka.raft.VoterSetTest;
|
||||
import org.apache.kafka.raft.internals.StringSerde;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
import org.apache.kafka.raft.internals.VoterSetTest;
|
||||
import org.apache.kafka.server.common.KRaftVersion;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
|
||||
|
|
Loading…
Reference in New Issue