Description
Seen killing threads in soak. During handleAssignment we call #suspend and #prepareCommit on all revoked tasks. Only RUNNING tasks are transitioned to SUSPENDED in #suspend, but #prepareCommit will assert that the state is either RUNNING or SUSPENDED.
So tasks that get revoked while in CREATED will hit an IllegalStateException during prepareCommit:
[2020-06-11T00:39:57-07:00] (streams-soak-KAFKA-10144-corrupted-standby-fix-eos-beta-broker-trunk_soak_i-09ef677ab88c040c8_streamslog) [2020-06-11 07:39:56,852] ERROR [stream-soak-test-90c4a6b9-e730-4211-9edb-146b74c54c8f-StreamThread-2] stream-thread [stream-soak-test-90c4a6b9-e730-4211-9edb-146b74c54c8f-StreamThread-2] Failed to close task 1_2 cleanly. Attempting to close remaining tasks before re-throwing: (org.apache.kafka.streams.processor.internals.TaskManager)[2020-06-11T00:39:57-07:00] (streams-soak-KAFKA-10144-corrupted-standby-fix-eos-beta-broker-trunk_soak_i-09ef677ab88c040c8_streamslog) [2020-06-11 07:39:56,852] ERROR [stream-soak-test-90c4a6b9-e730-4211-9edb-146b74c54c8f-StreamThread-2] stream-thread [stream-soak-test-90c4a6b9-e730-4211-9edb-146b74c54c8f-StreamThread-2] Failed to close task 1_2 cleanly. Attempting to close remaining tasks before re-throwing: (org.apache.kafka.streams.processor.internals.TaskManager)[2020-06-11T00:39:57-07:00] (streams-soak-KAFKA-10144-corrupted-standby-fix-eos-beta-broker-trunk_soak_i-09ef677ab88c040c8_streamslog) java.lang.IllegalStateException: Illegal state CREATED while preparing standby task 1_2 for committing at org.apache.kafka.streams.processor.internals.StandbyTask.prepareCommit(StandbyTask.java:142) at org.apache.kafka.streams.processor.internals.TaskManager.handleAssignment(TaskManager.java:228) at org.apache.kafka.streams.processor.internals.StreamsPartitionAssignor.onAssignment(StreamsPartitionAssignor.java:1350) at org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.invokeOnAssignment(ConsumerCoordinator.java:279) at org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.onJoinComplete(ConsumerCoordinator.java:421) at org.apache.kafka.clients.consumer.internals.AbstractCoordinator.joinGroupIfNeeded(AbstractCoordinator.java:440) at org.apache.kafka.clients.consumer.internals.AbstractCoordinator.ensureActiveGroup(AbstractCoordinator.java:359) at org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.poll(ConsumerCoordinator.java:506) at org.apache.kafka.clients.consumer.KafkaConsumer.updateAssignmentMetadataIfNeeded(KafkaConsumer.java:1263) at org.apache.kafka.clients.consumer.KafkaConsumer.poll(KafkaConsumer.java:1229) at org.apache.kafka.clients.consumer.KafkaConsumer.poll(KafkaConsumer.java:1204) at org.apache.kafka.streams.processor.internals.StreamThread.pollRequests(StreamThread.java:763) at org.apache.kafka.streams.processor.internals.StreamThread.runOnce(StreamThread.java:623) at org.apache.kafka.streams.processor.internals.StreamThread.runLoop(StreamThread.java:550) at org.apache.kafka.streams.processor.internals.StreamThread.run(StreamThread.java:509)
Attachments
Issue Links
- is duplicated by
-
KAFKA-10152 Attempt to write checkpoint without first committing during recycle
- Resolved
- links to