guozhangwang commented on a change in pull request #9569: URL: https://github.com/apache/kafka/pull/9569#discussion_r522675518
########## File path: clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java ########## @@ -713,6 +715,8 @@ public void sendOffsetsToTransaction(Map<TopicPartition, OffsetAndMetadata> offs * does not support transactions (i.e. if its version is lower than 0.11.0.0) * @throws org.apache.kafka.common.errors.AuthorizationException fatal error indicating that the configured * transactional.id is not authorized. See the exception for more details + * @throws org.apache.kafka.common.errors.InvalidProducerEpochException if the producer has attempted to produce with an old epoch Review comment: For both these two functions: are we certain that InvalidProducerEpochException could be thrown directly from the call? For sendOffsetsToTransaction we are sending a commit-offset request to the consumer coordinator which should not get this error code, and for both AddPartitionsToTxn and EndTxn responses we should only have `INVALID_PRODUCER_EPOCH` error for version <= 1, right? Even if they are thrown, then we should add them in the `throws` as well similar to ProducerFencedException, right? ########## File path: core/src/main/scala/kafka/log/ProducerStateManager.scala ########## @@ -203,7 +203,7 @@ private[log] class ProducerAppendInfo(val topicPartition: TopicPartition, if (origin == AppendOrigin.Replication) { warn(message) } else { - throw new ProducerFencedException(message) + throw new InvalidProducerEpochException(message) Review comment: Let's add a comment why we change this to the InvalidProducerEpochException. ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org