mirror of https://github.com/apache/kafka.git
KAFKA-10787 Apply spotless to raft module (#16278)
Reviewers: Chia-Ping Tsai <chia7712@gmail.com>
This commit is contained in:
parent
817da3fb5d
commit
d239dde8f6
|
@ -205,7 +205,6 @@ def excludedSpotlessModules = [':clients',
|
|||
':connect:runtime',
|
||||
':core',
|
||||
':metadata',
|
||||
':raft',
|
||||
':server',
|
||||
':storage',
|
||||
':storage:storage-api', // rename in settings.gradle
|
||||
|
|
|
@ -16,19 +16,21 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.HashMap;
|
||||
import java.util.Map;
|
||||
import java.util.Optional;
|
||||
import java.util.Set;
|
||||
import java.util.stream.Collectors;
|
||||
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;
|
||||
|
||||
import java.util.HashMap;
|
||||
import java.util.Map;
|
||||
import java.util.Optional;
|
||||
import java.util.Set;
|
||||
import java.util.stream.Collectors;
|
||||
|
||||
public class CandidateState implements EpochState {
|
||||
private final int localId;
|
||||
private final Uuid localDirectoryId;
|
||||
|
|
|
@ -16,7 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.Objects;
|
||||
import org.apache.kafka.common.message.KRaftVersionRecord;
|
||||
import org.apache.kafka.common.message.LeaderChangeMessage;
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
|
@ -25,6 +24,8 @@ import org.apache.kafka.common.message.VotersRecord;
|
|||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
import org.apache.kafka.common.record.ControlRecordType;
|
||||
|
||||
import java.util.Objects;
|
||||
|
||||
public final class ControlRecord {
|
||||
private final ControlRecordType recordType;
|
||||
private final ApiMessage message;
|
||||
|
|
|
@ -16,15 +16,16 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import java.util.List;
|
||||
import java.util.Objects;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalInt;
|
||||
import java.util.Set;
|
||||
import java.util.stream.Collectors;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
/**
|
||||
* Encapsulate election state stored on disk after every state change.
|
||||
|
|
|
@ -16,9 +16,10 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
import java.io.Closeable;
|
||||
import java.util.Optional;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
|
||||
public interface EpochState extends Closeable {
|
||||
|
||||
|
|
|
@ -16,13 +16,15 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.generated.QuorumStateDataJsonConverter;
|
||||
|
||||
import com.fasterxml.jackson.databind.JsonNode;
|
||||
import com.fasterxml.jackson.databind.ObjectMapper;
|
||||
import com.fasterxml.jackson.databind.node.ObjectNode;
|
||||
import com.fasterxml.jackson.databind.node.ShortNode;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
import org.apache.kafka.raft.generated.QuorumStateDataJsonConverter;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
|
|
@ -16,17 +16,19 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.Set;
|
||||
import org.apache.kafka.common.Node;
|
||||
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;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.Set;
|
||||
|
||||
public class FollowerState implements EpochState {
|
||||
private final int fetchTimeoutMs;
|
||||
private final int epoch;
|
||||
|
|
|
@ -37,6 +37,7 @@ import org.apache.kafka.common.requests.VoteRequest;
|
|||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.server.util.InterBrokerSendThread;
|
||||
import org.apache.kafka.server.util.RequestAndCompletionHandler;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
|
|
@ -82,6 +82,7 @@ import org.apache.kafka.snapshot.RecordsSnapshotReader;
|
|||
import org.apache.kafka.snapshot.RecordsSnapshotWriter;
|
||||
import org.apache.kafka.snapshot.SnapshotReader;
|
||||
import org.apache.kafka.snapshot.SnapshotWriter;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.net.InetSocketAddress;
|
||||
|
|
|
@ -21,6 +21,7 @@ import org.apache.kafka.common.requests.RequestHeader;
|
|||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.server.fault.FaultHandler;
|
||||
import org.apache.kafka.server.util.ShutdownableThread;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.concurrent.CompletableFuture;
|
||||
|
|
|
@ -28,6 +28,7 @@ import org.apache.kafka.common.utils.Timer;
|
|||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.ArrayList;
|
||||
|
|
|
@ -16,15 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.UncheckedIOException;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalInt;
|
||||
import java.util.Random;
|
||||
import java.util.function.Supplier;
|
||||
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.network.ListenerName;
|
||||
|
@ -33,8 +24,18 @@ import org.apache.kafka.common.utils.Time;
|
|||
import org.apache.kafka.raft.internals.BatchAccumulator;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.raft.internals.VoterSet;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.UncheckedIOException;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalInt;
|
||||
import java.util.Random;
|
||||
import java.util.function.Supplier;
|
||||
|
||||
/**
|
||||
* This class is responsible for managing the current state of this node and ensuring
|
||||
* only valid state transitions. Below we define the possible state transitions and
|
||||
|
|
|
@ -16,8 +16,8 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
|
||||
import java.util.concurrent.CompletableFuture;
|
||||
|
||||
|
|
|
@ -16,14 +16,17 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalInt;
|
||||
import org.apache.kafka.common.KafkaException;
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.raft.errors.NotLeaderException;
|
||||
import org.apache.kafka.snapshot.SnapshotReader;
|
||||
import org.apache.kafka.snapshot.SnapshotWriter;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalInt;
|
||||
|
||||
import static java.util.Collections.singletonList;
|
||||
|
||||
public class ReplicatedCounter implements RaftClient.Listener<Integer> {
|
||||
|
|
|
@ -16,13 +16,14 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.Optional;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.record.Records;
|
||||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
import org.apache.kafka.snapshot.RawSnapshotWriter;
|
||||
|
||||
import java.util.Optional;
|
||||
|
||||
public interface ReplicatedLog extends AutoCloseable {
|
||||
|
||||
/**
|
||||
|
|
|
@ -16,6 +16,8 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.common.Node;
|
||||
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collection;
|
||||
import java.util.HashMap;
|
||||
|
@ -24,7 +26,6 @@ import java.util.Map;
|
|||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.Random;
|
||||
import org.apache.kafka.common.Node;
|
||||
|
||||
/**
|
||||
* The request manager keeps tracks of the connection with remote replicas.
|
||||
|
|
|
@ -20,6 +20,7 @@ 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;
|
||||
|
||||
import java.util.HashSet;
|
||||
|
|
|
@ -20,6 +20,7 @@ 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;
|
||||
|
||||
import java.util.Optional;
|
||||
|
|
|
@ -16,14 +16,16 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.Set;
|
||||
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;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.Set;
|
||||
|
||||
/**
|
||||
* The "voted" state is for voters who have cast their vote for a specific candidate.
|
||||
*
|
||||
|
|
|
@ -19,6 +19,9 @@ package org.apache.kafka.raft.internals;
|
|||
import org.apache.kafka.common.compress.Compression;
|
||||
import org.apache.kafka.common.errors.RecordBatchTooLargeException;
|
||||
import org.apache.kafka.common.memory.MemoryPool;
|
||||
import org.apache.kafka.common.message.LeaderChangeMessage;
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
||||
import org.apache.kafka.common.protocol.ObjectSerializationCache;
|
||||
import org.apache.kafka.common.record.MemoryRecords;
|
||||
import org.apache.kafka.common.record.MutableRecordBatch;
|
||||
|
@ -28,20 +31,17 @@ import org.apache.kafka.raft.errors.NotLeaderException;
|
|||
import org.apache.kafka.raft.errors.UnexpectedBaseOffsetException;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
|
||||
import org.apache.kafka.common.message.LeaderChangeMessage;
|
||||
import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
import java.io.Closeable;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collection;
|
||||
import java.util.Collections;
|
||||
import java.util.Iterator;
|
||||
import java.util.List;
|
||||
import java.util.Objects;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalInt;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.Iterator;
|
||||
import java.util.concurrent.ConcurrentLinkedQueue;
|
||||
import java.util.concurrent.atomic.AtomicLong;
|
||||
import java.util.concurrent.locks.ReentrantLock;
|
||||
|
|
|
@ -16,8 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import org.apache.kafka.common.message.KRaftVersionRecord;
|
||||
import org.apache.kafka.common.message.VotersRecord;
|
||||
import org.apache.kafka.common.utils.BufferSupplier;
|
||||
|
@ -31,8 +29,12 @@ import org.apache.kafka.server.common.serialization.RecordSerde;
|
|||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotReader;
|
||||
import org.apache.kafka.snapshot.SnapshotReader;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
|
||||
/**
|
||||
* The KRaft state machine for tracking control records in the topic partition.
|
||||
*
|
||||
|
|
|
@ -16,17 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.InputStream;
|
||||
import java.io.UncheckedIOException;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collections;
|
||||
import java.util.Iterator;
|
||||
import java.util.List;
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.Optional;
|
||||
import java.util.function.BiFunction;
|
||||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
import org.apache.kafka.common.protocol.ByteBufferAccessor;
|
||||
import org.apache.kafka.common.record.ControlRecordType;
|
||||
|
@ -43,6 +32,18 @@ import org.apache.kafka.raft.Batch;
|
|||
import org.apache.kafka.raft.ControlRecord;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.InputStream;
|
||||
import java.io.UncheckedIOException;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collections;
|
||||
import java.util.Iterator;
|
||||
import java.util.List;
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.Optional;
|
||||
import java.util.function.BiFunction;
|
||||
|
||||
public final class RecordsIterator<T> implements Iterator<Batch<T>>, AutoCloseable {
|
||||
private final Records records;
|
||||
private final RecordSerde<T> serde;
|
||||
|
|
|
@ -16,9 +16,10 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.apache.kafka.common.Uuid;
|
||||
|
||||
import java.util.Objects;
|
||||
import java.util.Optional;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
|
||||
public final class ReplicaKey {
|
||||
private final int id;
|
||||
|
|
|
@ -16,10 +16,10 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import java.util.Map;
|
||||
import java.util.NavigableMap;
|
||||
import java.util.Optional;
|
||||
import java.util.TreeMap;
|
||||
import java.util.Map;
|
||||
|
||||
/**
|
||||
* A implementation for {@code LogHistory} which uses a red-black tree to store values sorted by offset.
|
||||
|
|
|
@ -16,6 +16,13 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
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 java.net.InetSocketAddress;
|
||||
import java.util.Collections;
|
||||
import java.util.HashMap;
|
||||
|
@ -30,13 +37,6 @@ import java.util.function.Function;
|
|||
import java.util.stream.Collectors;
|
||||
import java.util.stream.Stream;
|
||||
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
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;
|
||||
|
||||
/**
|
||||
* A type for representing the set of voters for a topic partition.
|
||||
*
|
||||
|
|
|
@ -16,11 +16,12 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.util.function.Consumer;
|
||||
import org.apache.kafka.common.record.MemoryRecords;
|
||||
import org.apache.kafka.common.record.UnalignedMemoryRecords;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
|
||||
import java.util.function.Consumer;
|
||||
|
||||
/**
|
||||
* A thin facade for a {@code RawSnapshotWriter} that notifies a callback when freeze completes successfully.
|
||||
*/
|
||||
|
|
|
@ -17,9 +17,6 @@
|
|||
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
||||
import org.apache.kafka.common.record.ControlRecordType;
|
||||
import org.apache.kafka.common.utils.BufferSupplier;
|
||||
|
@ -28,6 +25,10 @@ import org.apache.kafka.raft.OffsetAndEpoch;
|
|||
import org.apache.kafka.raft.internals.RecordsIterator;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
|
||||
public final class RecordsSnapshotReader<T> implements SnapshotReader<T> {
|
||||
private final OffsetAndEpoch snapshotId;
|
||||
private final RecordsIterator<T> iterator;
|
||||
|
|
|
@ -28,8 +28,8 @@ 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.internals.BatchAccumulator.CompletedBatch;
|
||||
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.serialization.RecordSerde;
|
||||
|
||||
|
|
|
@ -16,9 +16,10 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.nio.file.Path;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
|
||||
import java.nio.file.Path;
|
||||
|
||||
public final class SnapshotPath {
|
||||
public final Path path;
|
||||
public final OffsetAndEpoch snapshotId;
|
||||
|
|
|
@ -16,8 +16,8 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
|
||||
import java.util.List;
|
||||
|
||||
|
|
|
@ -17,10 +17,11 @@
|
|||
package org.apache.kafka.snapshot;
|
||||
|
||||
import org.apache.kafka.common.utils.BufferSupplier;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.KafkaRaftClient;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.internals.IdentitySerde;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
|
|
|
@ -22,6 +22,7 @@ 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;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -23,10 +23,12 @@ import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
|||
import org.apache.kafka.common.message.VotersRecord;
|
||||
import org.apache.kafka.common.protocol.ApiMessage;
|
||||
import org.apache.kafka.common.record.ControlRecordType;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.mockito.Mockito;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
|
||||
public final class ControlRecordTest {
|
||||
@Test
|
||||
|
|
|
@ -16,17 +16,20 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Optional;
|
||||
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;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Optional;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertTrue;
|
||||
|
|
|
@ -16,8 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import com.fasterxml.jackson.databind.JsonNode;
|
||||
import com.fasterxml.jackson.databind.ObjectMapper;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.protocol.types.TaggedFields;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
|
@ -25,6 +23,13 @@ import org.apache.kafka.raft.generated.QuorumStateData;
|
|||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import com.fasterxml.jackson.databind.JsonNode;
|
||||
import com.fasterxml.jackson.databind.ObjectMapper;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
||||
import java.io.BufferedWriter;
|
||||
import java.io.File;
|
||||
import java.io.FileOutputStream;
|
||||
|
@ -35,9 +40,6 @@ import java.nio.charset.StandardCharsets;
|
|||
import java.util.Collections;
|
||||
import java.util.Optional;
|
||||
import java.util.Set;
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
|
|
|
@ -21,6 +21,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.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -49,6 +49,7 @@ 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.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
|
|
|
@ -18,6 +18,7 @@ package org.apache.kafka.raft;
|
|||
|
||||
import org.apache.kafka.common.utils.LogContext;
|
||||
import org.apache.kafka.server.fault.MockFaultHandler;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.mockito.Mockito;
|
||||
|
||||
|
|
|
@ -30,10 +30,11 @@ import org.apache.kafka.common.utils.Utils;
|
|||
import org.apache.kafka.raft.internals.StringSerde;
|
||||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
import org.apache.kafka.snapshot.RawSnapshotWriter;
|
||||
import org.apache.kafka.snapshot.SnapshotWriter;
|
||||
import org.apache.kafka.snapshot.SnapshotReader;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotWriter;
|
||||
import org.apache.kafka.snapshot.SnapshotReader;
|
||||
import org.apache.kafka.snapshot.SnapshotWriter;
|
||||
import org.apache.kafka.snapshot.SnapshotWriterReaderTest;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
@ -44,16 +45,16 @@ import java.util.Arrays;
|
|||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Optional;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.OptionalInt;
|
||||
import java.util.OptionalLong;
|
||||
import java.util.Set;
|
||||
import java.util.concurrent.atomic.AtomicLong;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertInstanceOf;
|
||||
import static org.junit.jupiter.api.Assertions.assertTrue;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
import static org.junit.jupiter.api.Assertions.assertTrue;
|
||||
|
||||
public final class KafkaRaftClientSnapshotTest {
|
||||
@Test
|
||||
|
|
|
@ -24,9 +24,9 @@ import org.apache.kafka.common.message.BeginQuorumEpochResponseData;
|
|||
import org.apache.kafka.common.message.DescribeQuorumResponseData;
|
||||
import org.apache.kafka.common.message.DescribeQuorumResponseData.ReplicaState;
|
||||
import org.apache.kafka.common.message.EndQuorumEpochResponseData;
|
||||
import org.apache.kafka.common.message.FetchRequestData;
|
||||
import org.apache.kafka.common.message.FetchResponseData;
|
||||
import org.apache.kafka.common.message.VoteResponseData;
|
||||
import org.apache.kafka.common.message.FetchRequestData;
|
||||
import org.apache.kafka.common.metrics.KafkaMetric;
|
||||
import org.apache.kafka.common.metrics.Metrics;
|
||||
import org.apache.kafka.common.protocol.ApiKeys;
|
||||
|
@ -46,6 +46,7 @@ import org.apache.kafka.raft.errors.NotLeaderException;
|
|||
import org.apache.kafka.raft.errors.UnexpectedBaseOffsetException;
|
||||
import org.apache.kafka.raft.internals.ReplicaKey;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -23,6 +23,7 @@ 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.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.raft;
|
|||
import org.apache.kafka.common.errors.TimeoutException;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.concurrent.CompletableFuture;
|
||||
|
|
|
@ -33,6 +33,7 @@ import org.apache.kafka.snapshot.MockRawSnapshotReader;
|
|||
import org.apache.kafka.snapshot.MockRawSnapshotWriter;
|
||||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
import org.apache.kafka.snapshot.RawSnapshotWriter;
|
||||
|
||||
import org.slf4j.Logger;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
|
|
@ -31,6 +31,7 @@ import org.apache.kafka.common.utils.LogContext;
|
|||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.snapshot.RawSnapshotReader;
|
||||
import org.apache.kafka.snapshot.RawSnapshotWriter;
|
||||
|
||||
import org.junit.jupiter.api.AfterEach;
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
|
|
@ -16,10 +16,11 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
|
||||
import java.nio.file.FileSystems;
|
||||
import java.nio.file.Path;
|
||||
import java.util.Optional;
|
||||
import org.apache.kafka.raft.generated.QuorumStateData;
|
||||
|
||||
public class MockQuorumStateStore implements QuorumStateStore {
|
||||
private Optional<QuorumStateData> current = Optional.empty();
|
||||
|
|
|
@ -25,6 +25,7 @@ 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.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
import org.mockito.Mockito;
|
||||
|
|
|
@ -16,11 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft;
|
||||
|
||||
import net.jqwik.api.AfterFailureMode;
|
||||
import net.jqwik.api.ForAll;
|
||||
import net.jqwik.api.Property;
|
||||
import net.jqwik.api.Tag;
|
||||
import net.jqwik.api.constraints.IntRange;
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
|
@ -42,6 +37,12 @@ import org.apache.kafka.server.common.serialization.RecordSerde;
|
|||
import org.apache.kafka.snapshot.RecordsSnapshotReader;
|
||||
import org.apache.kafka.snapshot.SnapshotReader;
|
||||
|
||||
import net.jqwik.api.AfterFailureMode;
|
||||
import net.jqwik.api.ForAll;
|
||||
import net.jqwik.api.Property;
|
||||
import net.jqwik.api.Tag;
|
||||
import net.jqwik.api.constraints.IntRange;
|
||||
|
||||
import java.net.InetSocketAddress;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.ArrayList;
|
||||
|
|
|
@ -18,6 +18,7 @@ package org.apache.kafka.raft;
|
|||
|
||||
import org.apache.kafka.common.Node;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.List;
|
||||
|
@ -28,8 +29,8 @@ import java.util.stream.IntStream;
|
|||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertTrue;
|
||||
import static org.junit.jupiter.api.Assertions.assertNotEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertTrue;
|
||||
|
||||
public class RequestManagerTest {
|
||||
private final MockTime time = new MockTime();
|
||||
|
|
|
@ -20,6 +20,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.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -20,6 +20,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.ReplicaKey;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -20,12 +20,13 @@ 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;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.Collections;
|
||||
import java.util.Optional;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
|
|
|
@ -34,6 +34,7 @@ import org.apache.kafka.common.utils.ByteUtils;
|
|||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.errors.UnexpectedBaseOffsetException;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -22,6 +22,7 @@ import org.apache.kafka.common.record.MemoryRecords;
|
|||
import org.apache.kafka.common.record.MutableRecordBatch;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.EnumSource;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -16,11 +16,12 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.Collections;
|
||||
import java.util.IdentityHashMap;
|
||||
import java.util.Set;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
|
|
|
@ -17,6 +17,7 @@
|
|||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.apache.kafka.raft.RaftMessage;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.mockito.Mockito;
|
||||
|
||||
|
|
|
@ -16,8 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.stream.IntStream;
|
||||
import org.apache.kafka.common.TopicPartition;
|
||||
import org.apache.kafka.common.Uuid;
|
||||
import org.apache.kafka.common.message.KRaftVersionRecord;
|
||||
|
@ -28,7 +26,12 @@ import org.apache.kafka.raft.MockLog;
|
|||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotWriter;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Optional;
|
||||
import java.util.stream.IntStream;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
||||
final class KRaftControlRecordStateMachineTest {
|
||||
|
|
|
@ -26,6 +26,7 @@ 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.junit.jupiter.api.AfterEach;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
|
@ -18,6 +18,7 @@ package org.apache.kafka.raft.internals;
|
|||
|
||||
import org.apache.kafka.raft.Batch;
|
||||
import org.apache.kafka.raft.BatchReader;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.mockito.Mockito;
|
||||
|
||||
|
|
|
@ -26,6 +26,7 @@ import org.apache.kafka.common.utils.BufferSupplier;
|
|||
import org.apache.kafka.raft.BatchReader;
|
||||
import org.apache.kafka.raft.ControlRecord;
|
||||
import org.apache.kafka.raft.internals.RecordsIteratorTest.TestBatch;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.EnumSource;
|
||||
|
|
|
@ -16,24 +16,6 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
import java.util.IdentityHashMap;
|
||||
import java.util.List;
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.Objects;
|
||||
import java.util.Optional;
|
||||
import java.util.Random;
|
||||
import java.util.Set;
|
||||
import java.util.concurrent.atomic.AtomicReference;
|
||||
import java.util.stream.Collectors;
|
||||
import java.util.stream.IntStream;
|
||||
import java.util.stream.Stream;
|
||||
import net.jqwik.api.ForAll;
|
||||
import net.jqwik.api.Property;
|
||||
import org.apache.kafka.common.compress.Compression;
|
||||
import org.apache.kafka.common.errors.CorruptRecordException;
|
||||
import org.apache.kafka.common.message.KRaftVersionRecord;
|
||||
|
@ -62,12 +44,34 @@ import org.apache.kafka.server.common.serialization.RecordSerde;
|
|||
import org.apache.kafka.snapshot.MockRawSnapshotWriter;
|
||||
import org.apache.kafka.snapshot.RecordsSnapshotWriter;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import net.jqwik.api.ForAll;
|
||||
import net.jqwik.api.Property;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.Arguments;
|
||||
import org.junit.jupiter.params.provider.EnumSource;
|
||||
import org.junit.jupiter.params.provider.MethodSource;
|
||||
import org.mockito.Mockito;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.ArrayList;
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
import java.util.IdentityHashMap;
|
||||
import java.util.List;
|
||||
import java.util.NoSuchElementException;
|
||||
import java.util.Objects;
|
||||
import java.util.Optional;
|
||||
import java.util.Random;
|
||||
import java.util.Set;
|
||||
import java.util.concurrent.atomic.AtomicReference;
|
||||
import java.util.stream.Collectors;
|
||||
import java.util.stream.IntStream;
|
||||
import java.util.stream.Stream;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertDoesNotThrow;
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
|
|
|
@ -20,6 +20,7 @@ import org.apache.kafka.common.errors.NotLeaderOrFollowerException;
|
|||
import org.apache.kafka.common.errors.TimeoutException;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
import org.apache.kafka.raft.MockExpirationService;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.concurrent.CompletableFuture;
|
||||
|
|
|
@ -18,6 +18,7 @@ package org.apache.kafka.raft.internals;
|
|||
|
||||
import org.apache.kafka.common.metrics.MetricConfig;
|
||||
import org.apache.kafka.common.utils.MockTime;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
|
|
@ -16,8 +16,10 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import java.util.Optional;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.Optional;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
|
||||
|
|
|
@ -16,11 +16,13 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.HashMap;
|
||||
import java.util.Map;
|
||||
import java.util.Optional;
|
||||
import java.util.stream.IntStream;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
|
||||
|
|
|
@ -16,6 +16,14 @@
|
|||
*/
|
||||
package org.apache.kafka.raft.internals;
|
||||
|
||||
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.junit.jupiter.api.Test;
|
||||
|
||||
import java.net.InetSocketAddress;
|
||||
import java.util.Arrays;
|
||||
import java.util.Collections;
|
||||
|
@ -27,12 +35,7 @@ import java.util.function.Function;
|
|||
import java.util.stream.Collectors;
|
||||
import java.util.stream.IntStream;
|
||||
import java.util.stream.Stream;
|
||||
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.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.assertThrows;
|
||||
|
|
|
@ -17,16 +17,17 @@
|
|||
package org.apache.kafka.snapshot;
|
||||
|
||||
import org.apache.kafka.common.compress.Compression;
|
||||
import org.apache.kafka.common.utils.BufferSupplier.GrowableBufferSupplier;
|
||||
import org.apache.kafka.common.record.MemoryRecords;
|
||||
import org.apache.kafka.common.record.Record;
|
||||
import org.apache.kafka.common.record.RecordBatch;
|
||||
import org.apache.kafka.common.record.SimpleRecord;
|
||||
import org.apache.kafka.common.record.UnalignedFileRecords;
|
||||
import org.apache.kafka.common.record.UnalignedMemoryRecords;
|
||||
import org.apache.kafka.common.utils.BufferSupplier.GrowableBufferSupplier;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import org.junit.jupiter.api.AfterEach;
|
||||
import org.junit.jupiter.api.BeforeEach;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
|
|
@ -16,13 +16,14 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
import org.apache.kafka.common.record.MemoryRecords;
|
||||
import org.apache.kafka.common.record.Records;
|
||||
import org.apache.kafka.common.record.UnalignedMemoryRecords;
|
||||
import org.apache.kafka.common.record.UnalignedRecords;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
|
||||
public final class MockRawSnapshotReader implements RawSnapshotReader {
|
||||
private final OffsetAndEpoch snapshotId;
|
||||
private final MemoryRecords data;
|
||||
|
|
|
@ -16,13 +16,14 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.function.Consumer;
|
||||
import org.apache.kafka.common.record.MemoryRecords;
|
||||
import org.apache.kafka.common.record.UnalignedMemoryRecords;
|
||||
import org.apache.kafka.common.utils.ByteBufferOutputStream;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.function.Consumer;
|
||||
|
||||
public final class MockRawSnapshotWriter implements RawSnapshotWriter {
|
||||
private final ByteBufferOutputStream data = new ByteBufferOutputStream(0);
|
||||
private final OffsetAndEpoch snapshotId;
|
||||
|
|
|
@ -16,12 +16,15 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
import java.util.function.Consumer;
|
||||
import org.apache.kafka.common.record.MemoryRecords;
|
||||
import org.apache.kafka.common.record.UnalignedMemoryRecords;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.concurrent.atomic.AtomicBoolean;
|
||||
import java.util.function.Consumer;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
|
|
|
@ -17,11 +17,6 @@
|
|||
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.HashMap;
|
||||
import java.util.Optional;
|
||||
import java.util.concurrent.atomic.AtomicReference;
|
||||
import java.util.stream.IntStream;
|
||||
import org.apache.kafka.common.message.KRaftVersionRecord;
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
||||
|
@ -34,7 +29,15 @@ 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.serialization.RecordSerde;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.nio.ByteBuffer;
|
||||
import java.util.HashMap;
|
||||
import java.util.Optional;
|
||||
import java.util.concurrent.atomic.AtomicReference;
|
||||
import java.util.stream.IntStream;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
import static org.junit.jupiter.api.Assertions.assertFalse;
|
||||
import static org.junit.jupiter.api.Assertions.assertThrows;
|
||||
|
|
|
@ -16,6 +16,21 @@
|
|||
*/
|
||||
package org.apache.kafka.snapshot;
|
||||
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
||||
import org.apache.kafka.common.record.ControlRecordUtils;
|
||||
import org.apache.kafka.common.record.Record;
|
||||
import org.apache.kafka.common.record.RecordBatch;
|
||||
import org.apache.kafka.common.utils.BufferSupplier;
|
||||
import org.apache.kafka.common.utils.BufferSupplier.GrowableBufferSupplier;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.Batch;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.RaftClientTestContext;
|
||||
import org.apache.kafka.raft.internals.StringSerde;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collections;
|
||||
import java.util.Iterator;
|
||||
|
@ -24,19 +39,6 @@ import java.util.Optional;
|
|||
import java.util.OptionalInt;
|
||||
import java.util.Random;
|
||||
import java.util.Set;
|
||||
import org.apache.kafka.common.message.SnapshotFooterRecord;
|
||||
import org.apache.kafka.common.message.SnapshotHeaderRecord;
|
||||
import org.apache.kafka.common.record.ControlRecordUtils;
|
||||
import org.apache.kafka.common.record.Record;
|
||||
import org.apache.kafka.common.record.RecordBatch;
|
||||
import org.apache.kafka.common.utils.BufferSupplier.GrowableBufferSupplier;
|
||||
import org.apache.kafka.common.utils.BufferSupplier;
|
||||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.Batch;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.raft.RaftClientTestContext;
|
||||
import org.apache.kafka.raft.internals.StringSerde;
|
||||
import org.junit.jupiter.api.Test;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertDoesNotThrow;
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.kafka.snapshot;
|
|||
import org.apache.kafka.common.utils.Utils;
|
||||
import org.apache.kafka.raft.OffsetAndEpoch;
|
||||
import org.apache.kafka.test.TestUtils;
|
||||
|
||||
import org.junit.jupiter.api.Test;
|
||||
import org.junit.jupiter.params.ParameterizedTest;
|
||||
import org.junit.jupiter.params.provider.ValueSource;
|
||||
|
|
Loading…
Reference in New Issue