diff --git a/core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala b/core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala index ba48a63..e61ff07 100644 --- a/core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala +++ b/core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala @@ -32,7 +32,7 @@ import kafka.integration.KafkaServerTestHarness import kafka.utils.{TestZKUtils, ShutdownableThread, TestUtils} import org.apache.kafka.common.KafkaException -import org.apache.kafka.common.errors.{InvalidTopicException, NotEnoughReplicasException} +import org.apache.kafka.common.errors.{InvalidTopicException, NotEnoughReplicasException, NotEnoughReplicasAfterAppendException} import org.apache.kafka.clients.producer._ class ProducerFailureHandlingTest extends KafkaServerTestHarness { @@ -348,8 +348,9 @@ class ProducerFailureHandlingTest extends KafkaServerTestHarness { fail("Expected exception when producing to topic with fewer brokers than min.insync.replicas") } catch { case e: ExecutionException => - if (!e.getCause.isInstanceOf[NotEnoughReplicasException]) { - fail("Expected NotEnoughReplicasException when producing to topic with fewer brokers than min.insync.replicas") + if (!e.getCause.isInstanceOf[NotEnoughReplicasException] && + !e.getCause.isInstanceOf[NotEnoughReplicasAfterAppendException]) { + fail("Expected NotEnoughReplicasException or NotEnoughReplicasAfterAppendException when producing to topic with fewer brokers than min.insync.replicas") } }