kafka-features.sh
tool to upgrade to share.version=1
. For more information, please read the release notes.kafka.log.LogCleaner
to org.apache.kafka.storage.internals.log.LogCleaner
in the log4j2.yaml configuration file. Added loggers for org.apache.kafka.storage.internals.log.LogCleaner$CleanerThread
and org.apache.kafka.storage.internals.log.Cleaner
classes to CleanerAppender.state-change.log
files has been updated from stage-change.log.[date]
to state-change.log.[date]
in the log4j2.yaml configuration file.log.cleaner.enable
is deprecated. Users should no longer set it to false
to prepare for future removal. After the removal, log.cleaner.threads
will also have a lower bound of 1. For further details, please refer to KIP-1148.min.insync.replicas
value at the broker-level config will be removed. Please set at the cluster-level if necessary. For further details, please refer to here.flush
method now detects potential deadlocks and prohibits its use inside a callback. This change prevents unintended blocking behavior, which was a known risk in earlier versions.force
option of ConfigCommand
has been removed, as it has been non-operational since version 0.10.1.0.listConsumerGroups()
and listConsumerGroups(ListConsumerGroupsOptions)
methods in Admin
are deprecated, and will be removed in the next major version. Use Admin.listGroups(ListGroupsOptions.forConsumerGroups())
instead.window.size.ms
and window.inner.serde.class
in StreamsConfig
are deprecated. Use the corresponding string constants defined in TimeWindowedSerializer
, TimeWindowedDeserializer
, SessionWindowedSerializer
and SessionWindowedDeserializer
instead.For a rolling upgrade:
Note: Apache Kafka 4.0 only supports KRaft mode - ZooKeeper mode has been removed. As such, broker upgrades to 4.0.0 (and higher) require KRaft mode and the software and metadata versions must be at least 3.3.x (the first version when KRaft mode was deemed production ready). For clusters in KRaft mode with versions older than 3.3.x, we recommend upgrading to 3.9.x before upgrading to 4.0.x. Clusters in ZooKeeper mode have to be migrated to KRaft mode before they can be upgraded to 4.0.x.
For a rolling upgrade:
bin/kafka-features.sh --bootstrap-server localhost:9092 upgrade --release-version 4.0
IBP_4_0_IV1(23, "4.0", "IV1", true)
means this version has metadata changes). Given your current and target versions, a downgrade is only possible if there are no metadata changes in the versions between.state-change.log
files has been updated from stage-change.log.[date]
to state-change.log.[date]
in the log4j2.yaml configuration file. See KAFKA-19576 for details.group.coordinator
.Eligible Leader Replicas
(ELR). For more information check here.org.apache.kafka.sasl.oauthbearer.allowed.urls
) to set the allowed URLs as SASL OAUTHBEARER token or jwks endpoints. By default, the value is an empty list. Users should explicitly set the allowed list if necessary.metrics.jmx.blacklist
and metrics.jmx.whitelist
configurations were removed from the org.apache.kafka.common.metrics.JmxReporter
Please use metrics.jmx.exclude
and metrics.jmx.include
respectively instead.auto.include.jmx.reporter
configuration was removed. The metric.reporters
configuration is now set to org.apache.kafka.common.metrics.JmxReporter
by default.org.apache.kafka.common.metrics.JmxReporter
with string argument was removed. See KIP-606 for details.bufferpool-wait-time-total
, io-waittime-total
, and iotime-total
metrics were removed. Please use bufferpool-wait-time-ns-total
, io-wait-time-ns-total
, and io-time-ns-total
metrics as replacements, respectively.kafka.common.requests.DescribeLogDirsResponse.LogDirInfo
class was removed. Please use the kafka.clients.admin.DescribeLogDirsResult.descriptions()
class and kafka.clients.admin.DescribeLogDirsResult.allDescriptions()
instead.kafka.common.requests.DescribeLogDirsResponse.ReplicaInfo
class was removed. Please use the kafka.clients.admin.DescribeLogDirsResult.descriptions()
class and kafka.clients.admin.DescribeLogDirsResult.allDescriptions()
instead.org.apache.kafka.common.security.oauthbearer.secured.OAuthBearerLoginCallbackHandler
class was removed. Please use the org.apache.kafka.common.security.oauthbearer.OAuthBearerLoginCallbackHandler
class instead.org.apache.kafka.common.security.oauthbearer.secured.OAuthBearerValidatorCallbackHandler
class was removed. Please use the org.apache.kafka.common.security.oauthbearer.OAuthBearerValidatorCallbackHandler
class instead.org.apache.kafka.common.errors.NotLeaderForPartitionException
class was removed. The org.apache.kafka.common.errors.NotLeaderOrFollowerException
is returned if a request could not be processed because the broker is not the leader or follower for a topic partition.org.apache.kafka.clients.producer.internals.DefaultPartitioner
and org.apache.kafka.clients.producer.UniformStickyPartitioner
classes were removed.log.message.format.version
and message.format.version
configs were removed.onNewBatch
in org.apache.kafka.clients.producer.Partitioner
class was removed.config/kraft
directory since Zookeeper has been removed. These files have been consolidated with other configuration files. Now all configuration files are in config
directory.--bootstrap-server
only supports comma-separated value, such as host1:port1,host2:port2,...
. Providing other formats, like space-separated bootstrap servers (e.g., host1:port1 host2:port2 host3:port3
), will result in an exception, even though this was allowed in Apache Kafka versions prior to 4.0.delegation.token.master.key
configuration was removed. Please use delegation.token.secret.key
instead.offsets.commit.required.acks
configuration was removed. See KIP-1041 for details.log.message.timestamp.difference.max.ms
configuration was removed. Please use log.message.timestamp.before.max.ms
and log.message.timestamp.after.max.ms
instead. See KIP-937 for details.remote.log.manager.copier.thread.pool.size
configuration default value was changed to 10 from -1. Values of -1 are no longer valid; a minimum value of 1 or higher is required. See KIP-1030remote.log.manager.expiration.thread.pool.size
configuration default value was changed to 10 from -1. Values of -1 are no longer valid; a minimum value of 1 or higher is required. See KIP-1030remote.log.manager.thread.pool.size
configuration default value was changed to 2 from 10. See KIP-1030segment.bytes/log.segment.bytes
has changed from 14 bytes to 1MB. See KIP-1030use.incremental.alter.configs
configuration was removed from MirrorSourceConnector
. The modified behavior is now identical to the previous required
configuration, therefore users should ensure that brokers in the target cluster are at least running 2.3.0.add.source.alias.to.metrics
configuration was removed from MirrorSourceConnector
. The source cluster alias is now always added to the metrics.config.properties.blacklist
was removed from the org.apache.kafka.connect.mirror.MirrorSourceConfig
Please use config.properties.exclude
instead.topics.blacklist
was removed from the org.apache.kafka.connect.mirror.MirrorSourceConfig
Please use topics.exclude
instead.groups.blacklist
was removed from the org.apache.kafka.connect.mirror.MirrorSourceConfig
Please use groups.exclude
instead.kafka.common.MessageReader
class was removed. Please use the org.apache.kafka.tools.api.RecordReader
interface to build custom readers for the kafka-console-producer
tool.kafka.tools.DefaultMessageFormatter
class was removed. Please use the org.apache.kafka.tools.consumer.DefaultMessageFormatter
class instead.kafka.tools.LoggingMessageFormatter
class was removed. Please use the org.apache.kafka.tools.consumer.LoggingMessageFormatter
class instead.kafka.tools.NoOpMessageFormatter
class was removed. Please use the org.apache.kafka.tools.consumer.NoOpMessageFormatter
class instead.--whitelist
option was removed from the kafka-console-consumer
command line tool. Please use --include
instead.kafka.admin.FeatureCommand
, kafka.tools.ClusterTool
, kafka.tools.EndToEndLatency
, kafka.tools.StateChangeLogMerger
, kafka.tools.StreamsResetter
, kafka.tools.JmxTool
.--authorizer
, --authorizer-properties
, and --zk-tls-config-file
options were removed from the kafka-acls
command line tool. Please use --bootstrap-server
or --bootstrap-controller
instead.kafka.serializer.Decoder
trait was removed, please use the org.apache.kafka.tools.api.Decoder
interface to build custom decoders for the kafka-dump-log
tool.kafka.coordinator.group.OffsetsMessageFormatter
class was removed. Please use the org.apache.kafka.tools.consumer.OffsetsMessageFormatter
class instead.kafka.coordinator.group.GroupMetadataMessageFormatter
class was removed. Please use the org.apache.kafka.tools.consumer.GroupMetadataMessageFormatter
class instead.kafka.coordinator.transaction.TransactionLogMessageFormatter
class was removed. Please use the org.apache.kafka.tools.consumer.TransactionLogMessageFormatter
class instead.--topic-white-list
option was removed from the kafka-replica-verification
command line tool. Please use --topics-include
instead.--broker-list
option was removed from the kafka-verifiable-consumer
command line tool. Please use --bootstrap-server
instead.kafka-configs.sh
now uses incrementalAlterConfigs API to alter broker configurations instead of the deprecated alterConfigs API, and it will fall directly if the broker doesn’t support incrementalAlterConfigs API, which means the broker version is prior to 2.3.x. See KIP-1011 for more details.kafka.admin.ZkSecurityMigrator
tool was removed.whitelist
and blacklist
configurations were removed from the org.apache.kafka.connect.transforms.ReplaceField
transformation. Please use include
and exclude
respectively instead.onPartitionsRevoked(Collection<TopicPartition>)
and onPartitionsAssigned(Collection<TopicPartition>)
methods were removed from SinkTask
.commitRecord(SourceRecord)
method was removed from SourceTask
.poll(long)
method was removed from the consumer. Please use poll(Duration)
instead. Note that there is a difference in behavior between the two methods. The poll(Duration)
method does not block beyond the timeout awaiting partition assignment, whereas the earlier poll(long)
method used to wait beyond the timeout.committed(TopicPartition)
and committed(TopicPartition, Duration)
methods were removed from the consumer. Please use committed(Set<TopicPartition>)
and committed(Set<TopicPartition>, Duration)
instead.setException(KafkaException)
method was removed from the org.apache.kafka.clients.consumer.MockConsumer
. Please use setPollException(KafkaException)
instead.enable.idempotence
configuration will no longer automatically fall back when the max.in.flight.requests.per.connection
value exceeds 5.sendOffsetsToTransaction(Map<TopicPartition, OffsetAndMetadata>, String)
method has been removed from the Producer API.linger.ms
changed from 0 to 5 in Apache Kafka 4.0 as the efficiency gains from larger batches typically result in similar or lower producer latency despite the increased linger.alterConfigs
method was removed from the org.apache.kafka.clients.admin.Admin
. Please use incrementalAlterConfigs
instead.org.apache.kafka.common.ConsumerGroupState
enumeration and related methods have been deprecated. Please use GroupState
instead which applies to all types of group.Admin.describeConsumerGroups
method used to return a ConsumerGroupDescription
in state DEAD
if the group ID was not found. In Apache Kafka 4.0, the GroupIdNotFoundException
is thrown instead as part of the support for new types of group.org.apache.kafka.clients.admin.DeleteTopicsResult.values()
method was removed. Please use org.apache.kafka.clients.admin.DeleteTopicsResult.topicNameValues()
instead.org.apache.kafka.clients.admin.TopicListing.TopicListing(String, boolean)
method was removed. Please use org.apache.kafka.clients.admin.TopicListing.TopicListing(String, Uuid, boolean)
instead.org.apache.kafka.clients.admin.ListConsumerGroupOffsetsOptions.topicPartitions(List<TopicPartition>)
method was removed. Please use org.apache.kafka.clients.admin.Admin.listConsumerGroupOffsets(Map<String, ListConsumerGroupOffsetsSpec>, ListConsumerGroupOffsetsOptions)
instead.dryRun
methods were removed from the org.apache.kafka.clients.admin.UpdateFeaturesOptions
. Please use validateOnly
instead.org.apache.kafka.clients.admin.FeatureUpdate
with short and boolean arguments was removed. Please use the constructor that accepts short and the specified UpgradeType
enum instead.allowDowngrade
method was removed from the org.apache.kafka.clients.admin.FeatureUpdate
.org.apache.kafka.clients.admin.DescribeTopicsResult.DescribeTopicsResult(Map<String, KafkaFuture<TopicDescription>>)
method was removed. Please use org.apache.kafka.clients.admin.DescribeTopicsResult.DescribeTopicsResult(Map<Uuid, KafkaFuture<TopicDescription>>, Map<String, KafkaFuture<TopicDescription>>)
instead.values()
method was removed from the org.apache.kafka.clients.admin.DescribeTopicsResult
. Please use topicNameValues()
instead.all()
method was removed from the org.apache.kafka.clients.admin.DescribeTopicsResult
. Please use allTopicNames()
instead.JoinWindows.of()
and JoinWindows#grace()
. See KAFKA-17531 for details.KStream#transformValues()
which was removed with Apache Kafka 4.0.0 release, and you need to rewrite your program to use KStreams#processValues()
instead, pay close attention to the migration guide.--delete-config
option in the kafka-topics
command line tool has been deprecated.nextSegmentWithTxnIndex
is introduced in RLMM to allow the implementation to return the next segment metadata with a transaction index. This API is used when the consumers are enabled with isolation level as READ_COMMITTED. See KIP-1058 for more details.num.recovery.threads.per.data.dir
has been changed from 1 to 2. The impact of this is faster recovery post unclean shutdown at the expense of extra IO cycles. See KIP-1030message.timestamp.after.max.ms
has been changed from Long.Max to 1 hour. The impact of this messages with a timestamp of more than 1 hour in the future will be rejected when message.timestamp.type=CreateTime is set. See KIP-1030TransactionAbortableException
enhances error handling within transactional operations by clearly indicating scenarios where transactions should be aborted due to errors. It is important for applications to properly manage both TimeoutException
and TransactionAbortableException
when working with transaction producers.state-change.log
files incorrectly rotates to stage-change.log.[date]
(changing state to stage). This issue is corrected in 4.0.1. See KAFKA-19576 for details.See Upgrading From Previous Versions in the 3.9 documentation.
Was this page helpful?
Glad to hear it! Please tell us how we can improve.
Sorry to hear that. Please tell us how we can improve.