mirror of https://github.com/apache/kafka.git
KAFKA-14688 Move package org.apache.kafka.server.log.internals to org.apache.kafka.storage.internals.log (#13213)
Reviewers: Ismael Juma <ismael@juma.me.uk>
This commit is contained in:
parent
788793dee6
commit
1d3fb76092
|
@ -1043,7 +1043,7 @@ project(':core') {
|
|||
|
||||
task genTopicConfigDocs(type: JavaExec) {
|
||||
classpath = sourceSets.main.runtimeClasspath
|
||||
mainClass = 'org.apache.kafka.server.log.internals.LogConfig'
|
||||
mainClass = 'org.apache.kafka.storage.internals.log.LogConfig'
|
||||
if( !generatedDocsDir.exists() ) { generatedDocsDir.mkdirs() }
|
||||
standardOutput = new File(generatedDocsDir, "topic_config.html").newOutputStream()
|
||||
}
|
||||
|
|
|
@ -48,6 +48,7 @@
|
|||
<allow pkg="org.mockito"/>
|
||||
<allow pkg="kafka.security.authorizer"/>
|
||||
<allow pkg="org.apache.kafka.server"/>
|
||||
<allow pkg="org.apache.kafka.storage"/>
|
||||
<allow pkg="org.apache.kafka.clients"/>
|
||||
<allow pkg="org.apache.kafka.coordinator.group"/>
|
||||
<allow pkg="org.apache.kafka.metadata"/>
|
||||
|
|
|
@ -23,9 +23,9 @@ import kafka.server.BrokerTopicStats;
|
|||
import kafka.server.metadata.ConfigRepository;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.server.common.MetadataVersion;
|
||||
import org.apache.kafka.server.log.internals.CleanerConfig;
|
||||
import org.apache.kafka.server.log.internals.LogConfig;
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel;
|
||||
import org.apache.kafka.storage.internals.log.CleanerConfig;
|
||||
import org.apache.kafka.storage.internals.log.LogConfig;
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel;
|
||||
import org.apache.kafka.server.util.Scheduler;
|
||||
import scala.collection.JavaConverters;
|
||||
|
||||
|
|
|
@ -33,7 +33,7 @@ import kafka.server.ReplicaManager;
|
|||
import kafka.zk.KafkaZkClient;
|
||||
import org.apache.kafka.common.metrics.Metrics;
|
||||
import org.apache.kafka.common.utils.Time;
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel;
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel;
|
||||
import org.apache.kafka.server.util.Scheduler;
|
||||
import scala.compat.java8.OptionConverters;
|
||||
|
||||
|
|
|
@ -36,8 +36,8 @@ import org.apache.kafka.common.quota.{ClientQuotaAlteration, ClientQuotaEntity,
|
|||
import org.apache.kafka.common.security.JaasUtils
|
||||
import org.apache.kafka.common.security.scram.internals.{ScramCredentialUtils, ScramFormatter, ScramMechanism}
|
||||
import org.apache.kafka.common.utils.{Sanitizer, Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.util.{CommandDefaultOptions, CommandLineUtils}
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.zookeeper.client.ZKClientConfig
|
||||
|
||||
import scala.annotation.nowarn
|
||||
|
|
|
@ -30,8 +30,8 @@ import org.apache.kafka.common.config.ConfigResource
|
|||
import org.apache.kafka.common.errors.{ReplicaNotAvailableException, UnknownTopicOrPartitionException}
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.common.{KafkaException, KafkaFuture, TopicPartition, TopicPartitionReplica}
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.util.{CommandDefaultOptions, CommandLineUtils}
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
import scala.collection.{Map, Seq, mutable}
|
||||
|
|
|
@ -33,8 +33,8 @@ import org.apache.kafka.common.config.{ConfigResource, TopicConfig}
|
|||
import org.apache.kafka.common.errors.{ClusterAuthorizationException, TopicExistsException, UnsupportedVersionException}
|
||||
import org.apache.kafka.common.internals.Topic
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.util.{CommandDefaultOptions, CommandLineUtils}
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.annotation.nowarn
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -44,7 +44,7 @@ import org.apache.kafka.common.utils.Time
|
|||
import org.apache.kafka.common.{IsolationLevel, TopicPartition, Uuid}
|
||||
import org.apache.kafka.metadata.LeaderRecoveryState
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchDataInfo, FetchIsolation, FetchParams, LogOffsetMetadata}
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchDataInfo, FetchIsolation, FetchParams, LogOffsetMetadata}
|
||||
|
||||
import scala.collection.{Map, Seq}
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -20,7 +20,7 @@ package kafka.cluster
|
|||
import kafka.log.UnifiedLog
|
||||
import kafka.utils.Logging
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.server.log.internals.LogOffsetMetadata
|
||||
import org.apache.kafka.storage.internals.log.LogOffsetMetadata
|
||||
|
||||
import java.util.concurrent.atomic.AtomicReference
|
||||
|
||||
|
|
|
@ -46,8 +46,8 @@ import org.apache.kafka.common.{KafkaException, MessageFormatter, TopicPartition
|
|||
import org.apache.kafka.coordinator.group.generated.{GroupMetadataValue, OffsetCommitKey, OffsetCommitValue, GroupMetadataKey => GroupMetadataKeyData}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_1_IV0, IBP_2_1_IV0, IBP_2_1_IV1, IBP_2_3_IV0}
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchIsolation}
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchIsolation}
|
||||
|
||||
import scala.collection._
|
||||
import scala.collection.mutable.ArrayBuffer
|
||||
|
|
|
@ -36,9 +36,9 @@ import org.apache.kafka.common.requests.ProduceResponse.PartitionResponse
|
|||
import org.apache.kafka.common.requests.TransactionResult
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.common.{KafkaException, TopicPartition}
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchIsolation}
|
||||
import org.apache.kafka.server.record.BrokerCompressionType
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchIsolation}
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
import scala.collection.mutable
|
||||
|
|
|
@ -29,9 +29,9 @@ import org.apache.kafka.common.errors.{KafkaStorageException, OffsetOutOfRangeEx
|
|||
import org.apache.kafka.common.message.FetchResponseData
|
||||
import org.apache.kafka.common.record.MemoryRecords
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.LogFileUtils.offsetFromFileName
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, FetchDataInfo, LogConfig, LogDirFailureChannel, LogOffsetMetadata, OffsetPosition}
|
||||
import org.apache.kafka.storage.internals.log.LogFileUtils.offsetFromFileName
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, FetchDataInfo, LogConfig, LogDirFailureChannel, LogOffsetMetadata, OffsetPosition}
|
||||
|
||||
import java.util.{Collections, Optional}
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -32,7 +32,7 @@ import org.apache.kafka.common.record.MemoryRecords.RecordFilter
|
|||
import org.apache.kafka.common.record.MemoryRecords.RecordFilter.BatchRetention
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.{BufferSupplier, Time}
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, CleanerConfig, LastRecord, LogDirFailureChannel, OffsetMap, SkimpyOffsetMap, TransactionIndex}
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, CleanerConfig, LastRecord, LogDirFailureChannel, OffsetMap, SkimpyOffsetMap, TransactionIndex}
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
import scala.collection.mutable.ListBuffer
|
||||
|
|
|
@ -29,7 +29,7 @@ import kafka.utils.{Logging, Pool}
|
|||
import org.apache.kafka.common.{KafkaException, TopicPartition}
|
||||
import org.apache.kafka.common.errors.KafkaStorageException
|
||||
import org.apache.kafka.common.utils.Time
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel
|
||||
|
||||
import scala.collection.{Iterable, Seq, mutable}
|
||||
|
||||
|
|
|
@ -26,9 +26,9 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.errors.InvalidOffsetException
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.snapshot.Snapshots
|
||||
import org.apache.kafka.server.log.internals.{CorruptIndexException, LoadedLogOffsets, LogConfig, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{CorruptIndexException, LoadedLogOffsets, LogConfig, LogDirFailureChannel, LogOffsetMetadata}
|
||||
|
||||
import java.util.concurrent.{ConcurrentHashMap, ConcurrentMap}
|
||||
import scala.collection.{Set, mutable}
|
||||
|
|
|
@ -41,9 +41,9 @@ import org.apache.kafka.common.config.TopicConfig
|
|||
|
||||
import java.util.Properties
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.{CleanerConfig, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.log.internals.LogConfig.MessageFormatVersion
|
||||
import org.apache.kafka.storage.internals.log.LogConfig.MessageFormatVersion
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.log.{CleanerConfig, LogConfig, LogDirFailureChannel}
|
||||
|
||||
import scala.annotation.nowarn
|
||||
|
||||
|
|
|
@ -26,8 +26,8 @@ import org.apache.kafka.common.errors.CorruptRecordException
|
|||
import org.apache.kafka.common.record.FileRecords.{LogOffsetPosition, TimestampAndOffset}
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.{BufferSupplier, Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, AppendOrigin, CompletedTxn, LazyIndex, LogConfig, LogOffsetMetadata, OffsetIndex, OffsetPosition, TimeIndex, TimestampOffset, TransactionIndex, TxnIndexSearchResult, FetchDataInfo}
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, AppendOrigin, CompletedTxn, FetchDataInfo, LazyIndex, LogConfig, LogOffsetMetadata, OffsetIndex, OffsetPosition, TimeIndex, TimestampOffset, TransactionIndex, TxnIndexSearchResult}
|
||||
|
||||
import java.io.{File, IOException}
|
||||
import java.nio.file.attribute.FileTime
|
||||
|
|
|
@ -23,7 +23,7 @@ import org.apache.kafka.common.config.ConfigException
|
|||
import org.apache.kafka.common.protocol.types._
|
||||
import org.apache.kafka.common.record.RecordBatch
|
||||
import org.apache.kafka.common.utils.{ByteUtils, Crc32C, Time}
|
||||
import org.apache.kafka.server.log.internals._
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, BatchMetadata, CompletedTxn, CorruptSnapshotException, LogOffsetMetadata, ProducerAppendInfo, ProducerStateEntry, SnapshotFile, TxnMetadata}
|
||||
|
||||
import java.io.File
|
||||
import java.nio.ByteBuffer
|
||||
|
|
|
@ -40,12 +40,12 @@ import org.apache.kafka.common.utils.{PrimitiveRef, Time, Utils}
|
|||
import org.apache.kafka.common.{InvalidRecordException, KafkaException, TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.IBP_0_10_0_IV0
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, AppendOrigin, BatchMetadata, CompletedTxn, EpochEntry, FetchDataInfo, FetchIsolation, LastRecord, LogConfig, LogDirFailureChannel, LogOffsetMetadata, LogValidator, ProducerAppendInfo}
|
||||
import org.apache.kafka.server.log.remote.metadata.storage.TopicBasedRemoteLogMetadataManagerConfig
|
||||
import org.apache.kafka.server.record.BrokerCompressionType
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpointFile
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, AppendOrigin, BatchMetadata, CompletedTxn, EpochEntry, FetchDataInfo, FetchIsolation, LastRecord, LogConfig, LogDirFailureChannel, LogOffsetMetadata, LogValidator, ProducerAppendInfo}
|
||||
|
||||
import scala.annotation.nowarn
|
||||
import scala.collection.mutable.ListBuffer
|
||||
|
|
|
@ -22,9 +22,9 @@ import kafka.utils.{CoreUtils, Logging, ShutdownableThread}
|
|||
import org.apache.kafka.common.Uuid
|
||||
import org.apache.kafka.common.errors.CorruptRecordException
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.{LazyIndex, OffsetIndex, OffsetPosition, TimeIndex, TransactionIndex}
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteStorageManager.IndexType
|
||||
import org.apache.kafka.server.log.remote.storage.{RemoteLogSegmentMetadata, RemoteStorageManager}
|
||||
import org.apache.kafka.storage.internals.log.{LazyIndex, OffsetIndex, OffsetPosition, TimeIndex, TransactionIndex}
|
||||
|
||||
import java.io.{Closeable, File, InputStream}
|
||||
import java.nio.file.{Files, Path}
|
||||
|
|
|
@ -26,9 +26,10 @@ import org.apache.kafka.common.record.{ControlRecordUtils, MemoryRecords, Record
|
|||
import org.apache.kafka.common.utils.{BufferSupplier, Time}
|
||||
import org.apache.kafka.common.{KafkaException, TopicPartition, Uuid}
|
||||
import org.apache.kafka.raft.{Isolation, KafkaRaftClient, LogAppendInfo, LogFetchInfo, LogOffsetMetadata, OffsetAndEpoch, OffsetMetadata, ReplicatedLog, ValidOffsetAndEpoch}
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.snapshot.{FileRawSnapshotReader, FileRawSnapshotWriter, RawSnapshotReader, RawSnapshotWriter, SnapshotPath, Snapshots}
|
||||
import org.apache.kafka.storage.internals
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
|
||||
import java.io.File
|
||||
import java.nio.file.{Files, NoSuchFileException, Path}
|
||||
|
@ -191,7 +192,7 @@ final class KafkaMetadataLog private (
|
|||
override def updateHighWatermark(offsetMetadata: LogOffsetMetadata): Unit = {
|
||||
offsetMetadata.metadata.asScala match {
|
||||
case Some(segmentPosition: SegmentPosition) => log.updateHighWatermark(
|
||||
new org.apache.kafka.server.log.internals.LogOffsetMetadata(
|
||||
new internals.log.LogOffsetMetadata(
|
||||
offsetMetadata.offset,
|
||||
segmentPosition.baseOffset,
|
||||
segmentPosition.relativePosition)
|
||||
|
|
|
@ -44,11 +44,11 @@ import org.apache.kafka.raft
|
|||
import org.apache.kafka.raft.{RaftClient, RaftConfig}
|
||||
import org.apache.kafka.server.authorizer.Authorizer
|
||||
import org.apache.kafka.server.common.ApiMessageAndVersion
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteLogManagerConfig
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.server.util.{Deadline, FutureUtils, KafkaScheduler}
|
||||
import org.apache.kafka.snapshot.SnapshotWriter
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel
|
||||
|
||||
import java.net.InetAddress
|
||||
import java.util
|
||||
|
|
|
@ -33,8 +33,8 @@ import org.apache.kafka.common.config.internals.QuotaConfigs
|
|||
import org.apache.kafka.common.metrics.Quota
|
||||
import org.apache.kafka.common.metrics.Quota._
|
||||
import org.apache.kafka.common.utils.Sanitizer
|
||||
import org.apache.kafka.server.log.internals.{LogConfig, ThrottledReplicaListValidator}
|
||||
import org.apache.kafka.server.log.internals.LogConfig.MessageFormatVersion
|
||||
import org.apache.kafka.storage.internals.log.{LogConfig, ThrottledReplicaListValidator}
|
||||
import org.apache.kafka.storage.internals.log.LogConfig.MessageFormatVersion
|
||||
|
||||
import scala.annotation.nowarn
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -29,7 +29,7 @@ import org.apache.kafka.common.protocol.Errors
|
|||
import org.apache.kafka.common.requests.{ApiError, DescribeConfigsResponse}
|
||||
import org.apache.kafka.common.requests.DescribeConfigsResponse.ConfigSource
|
||||
import org.apache.kafka.server.config.ServerTopicConfigSynonyms
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.collection.{Map, mutable}
|
||||
import scala.compat.java8.OptionConverters._
|
||||
|
|
|
@ -24,7 +24,7 @@ import org.apache.kafka.common.config.ConfigResource.Type.{BROKER, TOPIC}
|
|||
import org.apache.kafka.controller.ConfigurationValidator
|
||||
import org.apache.kafka.common.errors.{InvalidConfigurationException, InvalidRequestException}
|
||||
import org.apache.kafka.common.internals.Topic
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.collection.mutable
|
||||
|
||||
|
|
|
@ -24,7 +24,7 @@ import org.apache.kafka.common.errors._
|
|||
import org.apache.kafka.common.protocol.Errors
|
||||
import org.apache.kafka.common.requests.FetchRequest.PartitionData
|
||||
import org.apache.kafka.common.requests.OffsetsForLeaderEpochResponse.{UNDEFINED_EPOCH, UNDEFINED_EPOCH_OFFSET}
|
||||
import org.apache.kafka.server.log.internals.{FetchIsolation, FetchParams, FetchPartitionData, LogOffsetMetadata}
|
||||
import org.apache.kafka.storage.internals.log.{FetchIsolation, FetchParams, FetchPartitionData, LogOffsetMetadata}
|
||||
|
||||
import scala.collection._
|
||||
|
||||
|
|
|
@ -36,7 +36,7 @@ import org.apache.kafka.common.network.{ListenerName, ListenerReconfigurable}
|
|||
import org.apache.kafka.common.security.authenticator.LoginManager
|
||||
import org.apache.kafka.common.utils.{ConfigUtils, Utils}
|
||||
import org.apache.kafka.server.config.ServerTopicConfigSynonyms
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.annotation.nowarn
|
||||
import scala.collection._
|
||||
|
|
|
@ -23,7 +23,7 @@ import org.apache.kafka.common.config.ConfigDef
|
|||
import org.apache.kafka.common.config.ConfigDef.Importance._
|
||||
import org.apache.kafka.common.config.ConfigDef.Range._
|
||||
import org.apache.kafka.common.config.ConfigDef.Type._
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
||||
|
|
|
@ -68,8 +68,8 @@ import org.apache.kafka.coordinator.group.GroupCoordinator
|
|||
import org.apache.kafka.server.authorizer._
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.{IBP_0_11_0_IV0, IBP_2_3_IV0}
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchIsolation, FetchParams, FetchPartitionData}
|
||||
import org.apache.kafka.server.record.BrokerCompressionType
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchIsolation, FetchParams, FetchPartitionData}
|
||||
|
||||
import java.lang.{Long => JLong}
|
||||
import java.nio.ByteBuffer
|
||||
|
|
|
@ -47,10 +47,10 @@ import org.apache.kafka.server.authorizer.Authorizer
|
|||
import org.apache.kafka.server.common.{MetadataVersion, MetadataVersionValidator}
|
||||
import org.apache.kafka.server.common.MetadataVersion._
|
||||
import org.apache.kafka.server.config.ServerTopicConfigSynonyms
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.log.internals.LogConfig.MessageFormatVersion
|
||||
import org.apache.kafka.storage.internals.log.LogConfig.MessageFormatVersion
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteLogManagerConfig
|
||||
import org.apache.kafka.server.record.BrokerCompressionType
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.zookeeper.client.ZKClientConfig
|
||||
|
||||
import scala.annotation.nowarn
|
||||
|
|
|
@ -31,8 +31,8 @@ import org.apache.kafka.metadata.KafkaConfigSchema
|
|||
import org.apache.kafka.metadata.bootstrap.{BootstrapDirectory, BootstrapMetadata}
|
||||
import org.apache.kafka.raft.RaftConfig
|
||||
import org.apache.kafka.server.config.ServerTopicConfigSynonyms
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import java.util.Optional
|
||||
import scala.collection.Seq
|
||||
|
|
|
@ -57,9 +57,9 @@ import org.apache.kafka.server.common.{ApiMessageAndVersion, MetadataVersion}
|
|||
import org.apache.kafka.server.common.MetadataVersion._
|
||||
import org.apache.kafka.server.fault.ProcessExitingFaultHandler
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteLogManagerConfig
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel
|
||||
import org.apache.zookeeper.client.ZKClientConfig
|
||||
|
||||
import scala.collection.{Map, Seq}
|
||||
|
|
|
@ -28,7 +28,7 @@ import org.apache.kafka.common.protocol.{ApiKeys, Errors}
|
|||
import org.apache.kafka.common.requests.OffsetsForLeaderEpochResponse.UNDEFINED_EPOCH
|
||||
import org.apache.kafka.common.requests.{FetchRequest, FetchResponse, RequestUtils}
|
||||
import org.apache.kafka.common.{TopicIdPartition, TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.log.internals.{FetchIsolation, FetchParams, FetchPartitionData}
|
||||
import org.apache.kafka.storage.internals.log.{FetchIsolation, FetchParams, FetchPartitionData}
|
||||
|
||||
import java.util
|
||||
import java.util.Optional
|
||||
|
|
|
@ -26,7 +26,7 @@ import kafka.utils.Logging
|
|||
import org.apache.kafka.common.Uuid
|
||||
import org.apache.kafka.common.errors.{InconsistentTopicIdException, KafkaStorageException}
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel
|
||||
|
||||
|
||||
object PartitionMetadataFile {
|
||||
|
|
|
@ -27,9 +27,9 @@ import org.apache.kafka.common.utils.Utils
|
|||
import org.apache.kafka.common.{KafkaException, TopicPartition}
|
||||
import org.apache.kafka.server.common.CheckpointFile.CheckpointReadBuffer
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.EpochEntry
|
||||
import org.apache.kafka.server.log.remote.storage.{RemoteLogSegmentMetadata, RemoteStorageException, RemoteStorageManager}
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpointFile
|
||||
import org.apache.kafka.storage.internals.log.EpochEntry
|
||||
|
||||
import java.io.{BufferedReader, File, InputStreamReader}
|
||||
import java.nio.charset.StandardCharsets
|
||||
|
|
|
@ -61,8 +61,8 @@ import org.apache.kafka.common.utils.Time
|
|||
import org.apache.kafka.image.{LocalReplicaChanges, MetadataImage, TopicsDelta}
|
||||
import org.apache.kafka.metadata.LeaderConstants.NO_LEADER
|
||||
import org.apache.kafka.server.common.MetadataVersion._
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchDataInfo, FetchParams, FetchPartitionData, LogConfig, LogDirFailureChannel, LogOffsetMetadata, RecordValidationException}
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchDataInfo, FetchParams, FetchPartitionData, LogConfig, LogDirFailureChannel, LogOffsetMetadata, RecordValidationException}
|
||||
|
||||
import java.nio.file.{Files, Paths}
|
||||
import java.util
|
||||
|
|
|
@ -48,7 +48,7 @@ import org.apache.kafka.common.requests.CreateTopicsRequest._
|
|||
import org.apache.kafka.common.requests.{AlterConfigsRequest, ApiError}
|
||||
import org.apache.kafka.common.security.scram.internals.{ScramCredentialUtils, ScramFormatter}
|
||||
import org.apache.kafka.common.utils.Sanitizer
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.collection.{Map, mutable, _}
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -18,8 +18,8 @@ package kafka.server.checkpoints
|
|||
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.server.common.CheckpointFile.EntryFormatter
|
||||
import org.apache.kafka.server.log.internals.{EpochEntry, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.checkpoint.CheckpointFileWithFailureHandler
|
||||
import org.apache.kafka.storage.internals.log.{EpochEntry, LogDirFailureChannel}
|
||||
|
||||
import java.io._
|
||||
import java.util.Optional
|
||||
|
|
|
@ -32,9 +32,9 @@ import org.apache.kafka.common.record._
|
|||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.metadata.MetadataRecordSerde
|
||||
import org.apache.kafka.metadata.bootstrap.BootstrapDirectory
|
||||
import org.apache.kafka.server.log.internals.{CorruptSnapshotException, OffsetIndex, TimeIndex, TransactionIndex}
|
||||
import org.apache.kafka.snapshot.Snapshots
|
||||
import org.apache.kafka.server.util.{CommandDefaultOptions, CommandLineUtils}
|
||||
import org.apache.kafka.storage.internals.log.{CorruptSnapshotException, OffsetIndex, TimeIndex, TransactionIndex}
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
import scala.collection.mutable
|
||||
|
|
|
@ -26,7 +26,7 @@ import kafka.utils.Implicits._
|
|||
import org.apache.kafka.common.{TopicPartition, Uuid}
|
||||
import org.apache.kafka.common.errors._
|
||||
import org.apache.kafka.common.internals.Topic
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.zookeeper.KeeperException.NodeExistsException
|
||||
|
||||
import scala.collection.{Map, Seq}
|
||||
|
|
|
@ -35,7 +35,7 @@ import org.apache.kafka.common.security.token.delegation.{DelegationToken, Token
|
|||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.common.{KafkaException, TopicPartition, Uuid}
|
||||
import org.apache.kafka.metadata.migration.ZkMigrationLeadershipState
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.zookeeper.KeeperException.{Code, NodeExistsException}
|
||||
import org.apache.zookeeper.OpResult.{CheckResult, CreateResult, ErrorResult, SetDataResult}
|
||||
import org.apache.zookeeper.client.ZKClientConfig
|
||||
|
|
|
@ -24,8 +24,8 @@ import org.apache.kafka.clients.admin.{Admin, AdminClientConfig, AlterConfigOp,
|
|||
import org.apache.kafka.common.config.{ConfigResource, TopicConfig}
|
||||
import org.apache.kafka.common.errors.{InvalidConfigurationException, InvalidRequestException, PolicyViolationException}
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.policy.AlterConfigPolicy
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.Assertions.{assertEquals, assertNull, assertTrue}
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, TestInfo, Timeout}
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -44,7 +44,7 @@ import org.apache.kafka.common.resource.{PatternType, ResourcePattern, ResourceT
|
|||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.common.{ConsumerGroupState, ElectionType, TopicCollection, TopicPartition, TopicPartitionInfo, TopicPartitionReplica, Uuid}
|
||||
import org.apache.kafka.controller.ControllerRequestContextUtil.ANONYMOUS_CONTEXT
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Disabled, TestInfo}
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -25,7 +25,7 @@ import org.apache.kafka.common.config.TopicConfig
|
|||
import org.apache.kafka.common.errors.{InvalidTimestampException, RecordTooLargeException, SerializationException, TimeoutException}
|
||||
import org.apache.kafka.common.record.{DefaultRecord, DefaultRecordBatch, Records, TimestampType}
|
||||
import org.apache.kafka.common.serialization.ByteArraySerializer
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Timeout
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -30,7 +30,7 @@ import org.apache.kafka.common.resource.ResourceType.{GROUP, TOPIC}
|
|||
import org.apache.kafka.common.resource.{PatternType, Resource, ResourcePattern, ResourcePatternFilter, ResourceType}
|
||||
import org.apache.kafka.common.security.auth.{KafkaPrincipal, SecurityProtocol}
|
||||
import org.apache.kafka.server.authorizer.Authorizer
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test, TestInfo}
|
||||
|
||||
|
|
|
@ -26,7 +26,7 @@ import org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEnd
|
|||
import org.apache.kafka.common.protocol.{ApiKeys, Errors}
|
||||
import org.apache.kafka.common.record.MemoryRecords
|
||||
import org.apache.kafka.common.requests.FetchRequest
|
||||
import org.apache.kafka.server.log.internals.{FetchDataInfo, FetchIsolation, FetchParams, FetchPartitionData, LogOffsetMetadata}
|
||||
import org.apache.kafka.storage.internals.log.{FetchDataInfo, FetchIsolation, FetchParams, FetchPartitionData, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Test
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.mockito.ArgumentMatchers.{any, anyInt}
|
||||
|
|
|
@ -59,9 +59,9 @@ import org.apache.kafka.common.requests.MetadataRequest
|
|||
import org.apache.kafka.common.security.auth.SecurityProtocol
|
||||
import org.apache.kafka.common.security.scram.ScramCredential
|
||||
import org.apache.kafka.common.serialization.{StringDeserializer, StringSerializer}
|
||||
import org.apache.kafka.server.log.internals.{CleanerConfig, LogConfig}
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.server.record.BrokerCompressionType
|
||||
import org.apache.kafka.storage.internals.log.{CleanerConfig, LogConfig}
|
||||
import org.apache.kafka.test.{TestSslUtils, TestUtils => JTestUtils}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Disabled, Test, TestInfo}
|
||||
|
|
|
@ -28,8 +28,8 @@ import org.apache.kafka.common.utils.Utils
|
|||
import org.apache.kafka.raft.internals.BatchBuilder
|
||||
import org.apache.kafka.raft._
|
||||
import org.apache.kafka.server.common.serialization.RecordSerde
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.snapshot.{FileRawSnapshotWriter, RawSnapshotReader, RawSnapshotWriter, SnapshotPath, Snapshots}
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.kafka.test.TestUtils.assertOptional
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
|
|
@ -27,8 +27,8 @@ import org.apache.kafka.common.protocol.{ApiKeys, Errors}
|
|||
import org.apache.kafka.common.record.{CompressionType, MemoryRecords, SimpleRecord}
|
||||
import org.apache.kafka.common.requests.LeaderAndIsrRequest
|
||||
import org.apache.kafka.common.requests.ProduceResponse.PartitionResponse
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.util.MockScheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.{BeforeEach, Test}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.mockito.Mockito.mock
|
||||
|
|
|
@ -26,7 +26,7 @@ import org.apache.kafka.clients.consumer.OffsetOutOfRangeException
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record.FileRecords
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.{FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.log.{FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
|
||||
/**
|
||||
* A stress test that instantiates a log and then runs continual appends against it from one thread and continual reads against it
|
||||
|
|
|
@ -29,9 +29,9 @@ import kafka.utils._
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.util.{KafkaScheduler, Scheduler}
|
||||
import org.apache.kafka.server.util.CommandLineUtils
|
||||
import org.apache.kafka.storage.internals.log.{LogConfig, LogDirFailureChannel}
|
||||
|
||||
import scala.math._
|
||||
|
||||
|
|
|
@ -15,7 +15,7 @@ package kafka.admin
|
|||
import kafka.server.{ConfigType, DynamicConfig, KafkaServer}
|
||||
import kafka.utils.TestUtils
|
||||
import kafka.zk.AdminZkClient
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
|
||||
import scala.collection.Seq
|
||||
|
||||
|
|
|
@ -34,7 +34,7 @@ import org.mockito.Mockito.{mock, when}
|
|||
import java.io.File
|
||||
import java.util.Properties
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.{CleanerConfig, LogConfig}
|
||||
import org.apache.kafka.storage.internals.log.{CleanerConfig, LogConfig}
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
||||
|
|
|
@ -35,8 +35,8 @@ import org.apache.kafka.common.requests.FetchRequest
|
|||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.common.{TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, CleanerConfig, FetchIsolation, FetchParams, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, CleanerConfig, FetchIsolation, FetchParams, LogConfig, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions.{assertEquals, assertFalse, assertTrue}
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.ArgumentMatchers
|
||||
|
|
|
@ -52,10 +52,10 @@ import org.apache.kafka.common.replica.ClientMetadata.DefaultClientMetadata
|
|||
import org.apache.kafka.common.security.auth.{KafkaPrincipal, SecurityProtocol}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.IBP_2_6_IV0
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, CleanerConfig, EpochEntry, FetchIsolation, FetchParams, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, CleanerConfig, EpochEntry, FetchIsolation, FetchParams, LogDirFailureChannel}
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
import org.junit.jupiter.params.provider.ValueSource
|
||||
|
||||
|
|
|
@ -19,7 +19,7 @@ package kafka.cluster
|
|||
import kafka.log.UnifiedLog
|
||||
import kafka.utils.MockTime
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.server.log.internals.LogOffsetMetadata
|
||||
import org.apache.kafka.storage.internals.log.LogOffsetMetadata
|
||||
import org.junit.jupiter.api.Assertions.{assertEquals, assertFalse, assertTrue}
|
||||
import org.junit.jupiter.api.{BeforeEach, Test}
|
||||
|
||||
|
|
|
@ -24,7 +24,7 @@ import kafka.zk.{KafkaZkClient, TopicPartitionStateZNode}
|
|||
import kafka.zookeeper._
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.server.common.MetadataVersion.{IBP_3_1_IV0, IBP_3_2_IV0}
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.zookeeper.KeeperException.Code
|
||||
import org.apache.zookeeper.data.Stat
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
|
|
|
@ -31,8 +31,8 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.protocol.Errors
|
||||
import org.apache.kafka.common.record.{MemoryRecords, RecordBatch, RecordConversionStats}
|
||||
import org.apache.kafka.common.requests.ProduceResponse.PartitionResponse
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, LogConfig}
|
||||
import org.apache.kafka.server.util.MockScheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, LogConfig}
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach}
|
||||
import org.mockito.Mockito.{CALLS_REAL_METHODS, mock, withSettings}
|
||||
|
||||
|
|
|
@ -37,8 +37,8 @@ import org.apache.kafka.clients.consumer.internals.ConsumerProtocol
|
|||
import org.apache.kafka.common.internals.Topic
|
||||
import org.apache.kafka.common.metrics.Metrics
|
||||
import org.apache.kafka.common.message.LeaveGroupRequestData.MemberIdentity
|
||||
import org.apache.kafka.server.log.internals.AppendOrigin
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.AppendOrigin
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -42,9 +42,9 @@ import org.apache.kafka.common.requests.ProduceResponse.PartitionResponse
|
|||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion._
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchDataInfo, FetchIsolation, LogOffsetMetadata}
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchDataInfo, FetchIsolation, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.{ArgumentCaptor, ArgumentMatchers}
|
||||
|
|
|
@ -34,7 +34,7 @@ import org.apache.kafka.common.record.{CompressionType, FileRecords, MemoryRecor
|
|||
import org.apache.kafka.common.requests._
|
||||
import org.apache.kafka.common.utils.{LogContext, MockTime, ProducerIdAndEpoch}
|
||||
import org.apache.kafka.common.{Node, TopicPartition}
|
||||
import org.apache.kafka.server.log.internals.{FetchDataInfo, FetchIsolation, LogConfig, LogOffsetMetadata}
|
||||
import org.apache.kafka.storage.internals.log.{FetchDataInfo, FetchIsolation, LogConfig, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.{ArgumentCaptor, ArgumentMatchers}
|
||||
|
|
|
@ -33,8 +33,8 @@ import org.apache.kafka.common.record._
|
|||
import org.apache.kafka.common.requests.ProduceResponse.PartitionResponse
|
||||
import org.apache.kafka.common.requests.TransactionResult
|
||||
import org.apache.kafka.common.utils.MockTime
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchDataInfo, FetchIsolation, LogConfig, LogOffsetMetadata}
|
||||
import org.apache.kafka.server.util.MockScheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchDataInfo, FetchIsolation, LogConfig, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.{ArgumentCaptor, ArgumentMatchers}
|
||||
|
|
|
@ -26,7 +26,7 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record.{CompressionType, MemoryRecords, RecordBatch}
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.{CleanerConfig, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.log.{CleanerConfig, LogConfig, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.{AfterEach, Tag}
|
||||
|
||||
import scala.collection.Seq
|
||||
|
|
|
@ -22,8 +22,8 @@ import kafka.utils._
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record.{CompressionType, MemoryRecords, RecordBatch, SimpleRecord}
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.{FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.record.BrokerCompressionType
|
||||
import org.apache.kafka.storage.internals.log.{FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api._
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -28,9 +28,9 @@ import org.apache.kafka.common.{KafkaException, TopicPartition}
|
|||
import org.apache.kafka.common.errors.KafkaStorageException
|
||||
import org.apache.kafka.common.record.{CompressionType, MemoryRecords, Record, SimpleRecord}
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{FetchDataInfo, LogConfig, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.junit.jupiter.api.Assertions.{assertFalse, _}
|
||||
import org.apache.kafka.storage.internals.log.{FetchDataInfo, LogConfig, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -26,7 +26,7 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, LogConfig, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, Test}
|
||||
|
||||
|
|
|
@ -26,7 +26,7 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_0_IV1, IBP_0_11_0_IV0, IBP_0_9_0}
|
||||
import org.apache.kafka.server.log.internals.{CleanerConfig, LogConfig}
|
||||
import org.apache.kafka.storage.internals.log.{CleanerConfig, LogConfig}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.extension.ExtensionContext
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -31,7 +31,7 @@ import org.apache.kafka.common.config.TopicConfig
|
|||
import org.apache.kafka.common.errors.CorruptRecordException
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.Utils
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, AppendOrigin, CleanerConfig, LogConfig, LogDirFailureChannel, OffsetMap}
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, AppendOrigin, CleanerConfig, LogConfig, LogDirFailureChannel, OffsetMap}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, Test}
|
||||
|
||||
|
|
|
@ -24,8 +24,8 @@ import kafka.utils.TestUtils
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record.SimpleRecord
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.{FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
||||
|
|
|
@ -27,7 +27,7 @@ import org.junit.jupiter.api.Test
|
|||
|
||||
import java.util.{Collections, Properties}
|
||||
import org.apache.kafka.server.common.MetadataVersion.IBP_3_0_IV1
|
||||
import org.apache.kafka.server.log.internals.{LogConfig, ThrottledReplicaListValidator}
|
||||
import org.apache.kafka.storage.internals.log.{LogConfig, ThrottledReplicaListValidator}
|
||||
|
||||
import scala.annotation.nowarn
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
|
|
@ -31,9 +31,9 @@ import org.apache.kafka.common.record.{CompressionType, ControlRecordType, Defau
|
|||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.IBP_0_11_0_IV0
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, CleanerConfig, EpochEntry, FetchDataInfo, LogConfig, LogDirFailureChannel, OffsetIndex, SnapshotFile}
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, CleanerConfig, EpochEntry, FetchDataInfo, LogConfig, LogDirFailureChannel, OffsetIndex, SnapshotFile}
|
||||
import org.junit.jupiter.api.Assertions.{assertDoesNotThrow, assertEquals, assertFalse, assertNotEquals, assertThrows, assertTrue}
|
||||
import org.junit.jupiter.api.function.Executable
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
|
|
@ -38,8 +38,8 @@ import java.io._
|
|||
import java.nio.file.Files
|
||||
import java.util.concurrent.{ConcurrentHashMap, ConcurrentMap, Future}
|
||||
import java.util.{Collections, Properties}
|
||||
import org.apache.kafka.server.log.internals.{FetchDataInfo, FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.storage.internals.log.{FetchDataInfo, FetchIsolation, LogConfig, LogDirFailureChannel}
|
||||
|
||||
import scala.collection.{Map, mutable}
|
||||
import scala.collection.mutable.ArrayBuffer
|
||||
|
|
|
@ -24,9 +24,9 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.{MockTime, Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{BatchMetadata, EpochEntry, LogConfig, ProducerStateEntry}
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpoint
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{BatchMetadata, EpochEntry, LogConfig, ProducerStateEntry}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
||||
|
|
|
@ -26,7 +26,6 @@ import kafka.utils.TestUtils
|
|||
import org.apache.kafka.common.Uuid
|
||||
import org.apache.kafka.common.record.{CompressionType, ControlRecordType, EndTransactionMarker, FileRecords, MemoryRecords, RecordBatch, SimpleRecord}
|
||||
import org.apache.kafka.common.utils.{Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, AppendOrigin, FetchDataInfo, FetchIsolation, LazyIndex, LogConfig, LogDirFailureChannel, TransactionIndex}
|
||||
import org.junit.jupiter.api.Assertions.{assertEquals, assertFalse}
|
||||
|
||||
import java.nio.file.Files
|
||||
|
@ -35,6 +34,7 @@ import kafka.log
|
|||
import org.apache.kafka.common.config.TopicConfig
|
||||
import org.apache.kafka.server.util.Scheduler
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpointFile
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, AppendOrigin, FetchDataInfo, FetchIsolation, LazyIndex, LogConfig, LogDirFailureChannel, TransactionIndex}
|
||||
|
||||
import scala.jdk.CollectionConverters._
|
||||
|
||||
|
|
|
@ -26,9 +26,9 @@ import org.apache.kafka.common.record._
|
|||
import org.apache.kafka.common.utils.{PrimitiveRef, Time}
|
||||
import org.apache.kafka.common.{InvalidRecordException, TopicPartition}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.LogValidator.ValidationResult
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, LogValidator, RecordValidationException}
|
||||
import org.apache.kafka.storage.internals.log.LogValidator.ValidationResult
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, LogValidator, RecordValidationException}
|
||||
import org.apache.kafka.test.TestUtils
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Test
|
||||
|
|
|
@ -28,7 +28,7 @@ import scala.collection._
|
|||
import scala.util.Random
|
||||
import kafka.utils.TestUtils
|
||||
import org.apache.kafka.common.errors.InvalidOffsetException
|
||||
import org.apache.kafka.server.log.internals.{OffsetIndex, OffsetPosition}
|
||||
import org.apache.kafka.storage.internals.log.{OffsetIndex, OffsetPosition}
|
||||
|
||||
import scala.annotation.nowarn
|
||||
|
||||
|
|
|
@ -19,7 +19,7 @@ package kafka.log
|
|||
|
||||
import java.nio._
|
||||
import kafka.utils.Exit
|
||||
import org.apache.kafka.server.log.internals.SkimpyOffsetMap
|
||||
import org.apache.kafka.storage.internals.log.SkimpyOffsetMap
|
||||
import org.junit.jupiter.api._
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
|
||||
|
|
|
@ -29,7 +29,7 @@ import org.apache.kafka.common.errors._
|
|||
import org.apache.kafka.common.internals.Topic
|
||||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.utils.{MockTime, Utils}
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, CompletedTxn, LogOffsetMetadata, ProducerAppendInfo, ProducerStateEntry, TxnMetadata}
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, CompletedTxn, LogOffsetMetadata, ProducerAppendInfo, ProducerStateEntry, TxnMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.Mockito.{mock, when}
|
||||
|
|
|
@ -20,7 +20,7 @@ package kafka.log
|
|||
import java.io.File
|
||||
import kafka.utils.TestUtils
|
||||
import org.apache.kafka.common.errors.InvalidOffsetException
|
||||
import org.apache.kafka.server.log.internals.{CorruptIndexException, TimeIndex, TimestampOffset}
|
||||
import org.apache.kafka.storage.internals.log.{CorruptIndexException, TimeIndex, TimestampOffset}
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.junit.jupiter.api.Assertions.{assertEquals, assertThrows}
|
||||
|
||||
|
|
|
@ -18,7 +18,7 @@ package kafka.log
|
|||
|
||||
import kafka.utils.TestUtils
|
||||
import org.apache.kafka.common.message.FetchResponseData
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, CorruptIndexException, TransactionIndex}
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, CorruptIndexException, TransactionIndex}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
||||
|
|
|
@ -31,12 +31,12 @@ import org.apache.kafka.common.record.MemoryRecords.RecordFilter
|
|||
import org.apache.kafka.common.record._
|
||||
import org.apache.kafka.common.requests.{ListOffsetsRequest, ListOffsetsResponse}
|
||||
import org.apache.kafka.common.utils.{BufferSupplier, Time, Utils}
|
||||
import org.apache.kafka.server.log.internals.{AbortedTxn, AppendOrigin, EpochEntry, FetchIsolation, LogConfig, LogOffsetMetadata, RecordValidationException}
|
||||
import org.apache.kafka.server.log.remote.metadata.storage.TopicBasedRemoteLogMetadataManagerConfig
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.server.util.{KafkaScheduler, Scheduler}
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpointFile
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{AbortedTxn, AppendOrigin, EpochEntry, FetchIsolation, LogConfig, LogOffsetMetadata, RecordValidationException}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.ArgumentMatchers
|
||||
|
|
|
@ -19,9 +19,9 @@ package kafka.log.remote
|
|||
import kafka.log.UnifiedLog
|
||||
import kafka.utils.MockTime
|
||||
import org.apache.kafka.common.{TopicIdPartition, TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.log.internals.{OffsetIndex, OffsetPosition, TimeIndex}
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteStorageManager.IndexType
|
||||
import org.apache.kafka.server.log.remote.storage.{RemoteLogSegmentId, RemoteLogSegmentMetadata, RemoteStorageManager}
|
||||
import org.apache.kafka.storage.internals.log.{OffsetIndex, OffsetPosition, TimeIndex}
|
||||
import org.apache.kafka.test.TestUtils
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
|
|
|
@ -24,11 +24,11 @@ import org.apache.kafka.common.config.AbstractConfig
|
|||
import org.apache.kafka.common.record.FileRecords.TimestampAndOffset
|
||||
import org.apache.kafka.common.record.{CompressionType, MemoryRecords, SimpleRecord}
|
||||
import org.apache.kafka.common.{KafkaException, TopicIdPartition, TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.log.internals._
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteStorageManager.IndexType
|
||||
import org.apache.kafka.server.log.remote.storage._
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpoint
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{EpochEntry, OffsetIndex, TimeIndex}
|
||||
import org.apache.kafka.test.TestUtils
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{BeforeEach, Test}
|
||||
|
|
|
@ -33,7 +33,7 @@ import org.apache.kafka.common.requests.{FetchRequest, FetchResponse}
|
|||
import org.apache.kafka.common.utils.Time
|
||||
import org.apache.kafka.server.metrics.KafkaYammerMetrics
|
||||
import org.apache.kafka.common.{KafkaException, TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.log.internals.LogOffsetMetadata
|
||||
import org.apache.kafka.storage.internals.log.LogOffsetMetadata
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Assumptions.assumeTrue
|
||||
import org.junit.jupiter.api.{BeforeEach, Test}
|
||||
|
|
|
@ -33,8 +33,8 @@ import org.apache.kafka.common.config.{ConfigException, SslConfigs}
|
|||
import org.apache.kafka.common.metrics.{JmxReporter, Metrics}
|
||||
import org.apache.kafka.common.network.ListenerName
|
||||
import org.apache.kafka.server.authorizer._
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.apache.kafka.test.MockMetricsReporter
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Test
|
||||
|
|
|
@ -40,7 +40,7 @@ import org.apache.kafka.common.quota.{ClientQuotaAlteration, ClientQuotaEntity}
|
|||
import org.apache.kafka.common.record.{CompressionType, RecordVersion}
|
||||
import org.apache.kafka.common.security.auth.KafkaPrincipal
|
||||
import org.apache.kafka.server.common.MetadataVersion.IBP_3_0_IV1
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{Test, Timeout}
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -27,8 +27,8 @@ import kafka.cluster.Partition
|
|||
import kafka.server.metadata.MockConfigRepository
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.common.record.SimpleRecord
|
||||
import org.apache.kafka.server.log.internals.{CleanerConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.{CleanerConfig, LogDirFailureChannel}
|
||||
|
||||
class HighwatermarkPersistenceTest {
|
||||
|
||||
|
|
|
@ -28,7 +28,7 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.metrics.Metrics
|
||||
import org.apache.kafka.common.utils.Time
|
||||
import org.apache.kafka.metadata.LeaderRecoveryState
|
||||
import org.apache.kafka.server.log.internals.{LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.apache.kafka.storage.internals.log.{LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.Mockito.{atLeastOnce, mock, verify, when}
|
||||
|
|
|
@ -93,7 +93,7 @@ import org.apache.kafka.common.message.OffsetDeleteResponseData.{OffsetDeleteRes
|
|||
import org.apache.kafka.coordinator.group.GroupCoordinator
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.{IBP_0_10_2_IV0, IBP_2_2_IV1}
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchParams, FetchPartitionData}
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchParams, FetchPartitionData}
|
||||
|
||||
class KafkaApisTest {
|
||||
private val requestChannel: RequestChannel = mock(classOf[RequestChannel])
|
||||
|
|
|
@ -38,8 +38,8 @@ import org.apache.kafka.common.Node
|
|||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.common.MetadataVersion.{IBP_0_8_2, IBP_3_0_IV1}
|
||||
import org.apache.kafka.server.config.ServerTopicConfigSynonyms
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.server.log.remote.storage.RemoteLogManagerConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.function.Executable
|
||||
|
||||
import scala.annotation.nowarn
|
||||
|
|
|
@ -31,7 +31,7 @@ import org.apache.kafka.common.record.MemoryRecords
|
|||
import org.apache.kafka.common.requests.{FetchRequest, UpdateMetadataRequest}
|
||||
import org.apache.kafka.common.{TopicIdPartition, TopicPartition, Uuid}
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.{FetchIsolation, FetchParams, FetchPartitionData}
|
||||
import org.apache.kafka.storage.internals.log.{FetchIsolation, FetchParams, FetchPartitionData}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Test
|
||||
import org.mockito.ArgumentMatchers.{any, anyBoolean}
|
||||
|
|
|
@ -38,7 +38,7 @@ import org.apache.kafka.common.{IsolationLevel, TopicIdPartition, TopicPartition
|
|||
import org.apache.kafka.image.{MetadataDelta, MetadataImage}
|
||||
import org.apache.kafka.metadata.LeaderRecoveryState
|
||||
import org.apache.kafka.metadata.PartitionRegistration
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchIsolation, FetchParams, FetchPartitionData, LogConfig, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchIsolation, FetchParams, FetchPartitionData, LogConfig, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, Test}
|
||||
import org.mockito.Mockito
|
||||
|
|
|
@ -29,8 +29,8 @@ import org.apache.kafka.common.requests.FetchRequest
|
|||
import org.apache.kafka.common.requests.FetchRequest.PartitionData
|
||||
import org.apache.kafka.common.{TopicIdPartition, TopicPartition, Uuid}
|
||||
import org.apache.kafka.metadata.LeaderRecoveryState
|
||||
import org.apache.kafka.server.log.internals.{FetchDataInfo, FetchIsolation, FetchParams, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.apache.kafka.server.util.KafkaScheduler
|
||||
import org.apache.kafka.storage.internals.log.{FetchDataInfo, FetchIsolation, FetchParams, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, Test}
|
||||
import org.mockito.ArgumentMatchers.{any, anyBoolean, anyInt, anyLong}
|
||||
|
|
|
@ -57,8 +57,8 @@ import org.apache.kafka.image.{AclsImage, ClientQuotasImage, ClusterImageTest, C
|
|||
import org.apache.kafka.metadata.LeaderConstants.NO_LEADER
|
||||
import org.apache.kafka.metadata.LeaderRecoveryState
|
||||
import org.apache.kafka.server.common.MetadataVersion.IBP_2_6_IV0
|
||||
import org.apache.kafka.server.log.internals.{AppendOrigin, FetchIsolation, FetchParams, FetchPartitionData, LogConfig, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.apache.kafka.server.util.MockScheduler
|
||||
import org.apache.kafka.storage.internals.log.{AppendOrigin, FetchIsolation, FetchParams, FetchPartitionData, LogConfig, LogDirFailureChannel, LogOffsetMetadata}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.junit.jupiter.params.ParameterizedTest
|
||||
|
|
|
@ -25,7 +25,7 @@ import kafka.utils.CoreUtils._
|
|||
import kafka.utils.TestUtils
|
||||
import org.apache.kafka.clients.producer.{KafkaProducer, ProducerRecord}
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.server.log.internals.LogConfig
|
||||
import org.apache.kafka.storage.internals.log.LogConfig
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, Test}
|
||||
|
||||
|
|
|
@ -17,8 +17,8 @@
|
|||
package kafka.server.checkpoints
|
||||
|
||||
import kafka.utils.{Logging, TestUtils}
|
||||
import org.apache.kafka.server.log.internals.{EpochEntry, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.checkpoint.LeaderEpochCheckpointFile
|
||||
import org.apache.kafka.storage.internals.log.{EpochEntry, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Test
|
||||
|
||||
|
|
|
@ -19,8 +19,8 @@ package kafka.server.checkpoints
|
|||
import kafka.utils.{Logging, TestUtils}
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.common.errors.KafkaStorageException
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel
|
||||
import org.apache.kafka.storage.internals.checkpoint.CheckpointFileWithFailureHandler
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Test
|
||||
import org.mockito.Mockito
|
||||
|
|
|
@ -29,8 +29,8 @@ import org.apache.kafka.common.TopicPartition
|
|||
import org.apache.kafka.common.record.RecordBatch
|
||||
import org.apache.kafka.common.serialization.ByteArrayDeserializer
|
||||
import org.apache.kafka.server.common.MetadataVersion
|
||||
import org.apache.kafka.server.log.internals.EpochEntry
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.EpochEntry
|
||||
import org.junit.jupiter.api.Assertions.{assertEquals, assertTrue}
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test, TestInfo}
|
||||
|
||||
|
|
|
@ -20,9 +20,9 @@ package kafka.server.epoch
|
|||
import kafka.utils.TestUtils
|
||||
import org.apache.kafka.common.TopicPartition
|
||||
import org.apache.kafka.common.requests.OffsetsForLeaderEpochResponse.{UNDEFINED_EPOCH, UNDEFINED_EPOCH_OFFSET}
|
||||
import org.apache.kafka.server.log.internals.{EpochEntry, LogDirFailureChannel}
|
||||
import org.apache.kafka.storage.internals.checkpoint.{LeaderEpochCheckpoint, LeaderEpochCheckpointFile}
|
||||
import org.apache.kafka.storage.internals.epoch.LeaderEpochFileCache
|
||||
import org.apache.kafka.storage.internals.log.{EpochEntry, LogDirFailureChannel}
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.Test
|
||||
|
||||
|
|
|
@ -29,7 +29,7 @@ import org.apache.kafka.common.metrics.Metrics
|
|||
import org.apache.kafka.common.protocol.Errors
|
||||
import org.apache.kafka.common.record.RecordBatch
|
||||
import org.apache.kafka.common.requests.OffsetsForLeaderEpochResponse.{UNDEFINED_EPOCH, UNDEFINED_EPOCH_OFFSET}
|
||||
import org.apache.kafka.server.log.internals.LogDirFailureChannel
|
||||
import org.apache.kafka.storage.internals.log.LogDirFailureChannel
|
||||
import org.junit.jupiter.api.Assertions._
|
||||
import org.junit.jupiter.api.{AfterEach, BeforeEach, Test}
|
||||
import org.mockito.Mockito.{mock, when}
|
||||
|
|
Some files were not shown because too many files have changed in this diff Show More
Loading…
Reference in New Issue