Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

KAFKA-10787: Apply spotless to raft module #16278

Merged
merged 6 commits into from
Jun 15, 2024
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
1 change: 0 additions & 1 deletion build.gradle
Original file line number Diff line number Diff line change
Expand Up @@ -205,7 +205,6 @@ def excludedSpotlessModules = [':clients',
':connect:runtime',
':core',
':metadata',
':raft',
':server',
':storage',
':storage:storage-api', // rename in settings.gradle
Expand Down
12 changes: 7 additions & 5 deletions raft/src/main/java/org/apache/kafka/raft/CandidateState.java
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand Down
3 changes: 2 additions & 1 deletion raft/src/main/java/org/apache/kafka/raft/ControlRecord.java
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand All @@ -25,6 +24,8 @@
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;
Expand Down
7 changes: 4 additions & 3 deletions raft/src/main/java/org/apache/kafka/raft/ElectionState.java
Original file line number Diff line number Diff line change
Expand Up @@ -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.
Expand Down
3 changes: 2 additions & 1 deletion raft/src/main/java/org/apache/kafka/raft/EpochState.java
Original file line number Diff line number Diff line change
Expand Up @@ -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 {

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -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;

Expand Down
8 changes: 5 additions & 3 deletions raft/src/main/java/org/apache/kafka/raft/FollowerState.java
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,6 +37,7 @@
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;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -82,6 +82,7 @@
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;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@
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;
Expand Down
1 change: 1 addition & 0 deletions raft/src/main/java/org/apache/kafka/raft/LeaderState.java
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@
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;
Expand Down
19 changes: 10 additions & 9 deletions raft/src/main/java/org/apache/kafka/raft/QuorumState.java
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand All @@ -33,8 +24,18 @@
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
Expand Down
2 changes: 1 addition & 1 deletion raft/src/main/java/org/apache/kafka/raft/RaftRequest.java
Original file line number Diff line number Diff line change
Expand Up @@ -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;

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -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> {
Expand Down
3 changes: 2 additions & 1 deletion raft/src/main/java/org/apache/kafka/raft/ReplicatedLog.java
Original file line number Diff line number Diff line change
Expand Up @@ -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 {

/**
Expand Down
3 changes: 2 additions & 1 deletion raft/src/main/java/org/apache/kafka/raft/RequestManager.java
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand All @@ -24,7 +26,6 @@
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.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,6 +20,7 @@
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;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -20,6 +20,7 @@
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;
Expand Down
6 changes: 4 additions & 2 deletions raft/src/main/java/org/apache/kafka/raft/VotedState.java
Original file line number Diff line number Diff line change
Expand Up @@ -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.
*
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,9 @@
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;
Expand All @@ -28,20 +31,17 @@
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;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand All @@ -31,8 +29,12 @@
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.
*
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand All @@ -43,6 +32,18 @@
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;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -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;
Expand Down
Loading