[ https://issues.apache.org/jira/browse/KAFKA-9618?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17047813#comment-17047813 ]
Boyang Chen edited comment on KAFKA-9618 at 3/1/20 2:15 AM: ------------------------------------------------------------ Reproduced the exception as: {code:java} [stream-soak-test-e6003ae8-1331-4b99-97e3-a3229aaa15fd-StreamThread-3] task [1_1] Failed to wiping state stores for task 1_1 (org.apache.kafka.streams.processor.internals.StreamTask) [2020-02-29T05:50:03-08:00] (streams-soak-trunk-eos_soak_i-0c3043df582d0379a_streamslog) [2020-02-29 13:49:58,304] ERROR [stream-soak-test-e6003ae8-1331-4b99-97e3-a3229aaa15fd-StreamThread-3] stream-thread [stream-soak-test-e6003ae8-1331-4b99-97e3-a3229aaa15fd-StreamThread-3] Encountered the following exception during processing and the thread is going to shut down: (org.apache.kafka.streams.processor.internals.StreamThread) [2020-02-29T05:50:03-08:00] (streams-soak-trunk-eos_soak_i-0c3043df582d0379a_streamslog) org.apache.kafka.streams.errors.TaskMigratedException: The deletion of directory for task 1_1failed; it means all tasks belonging to this thread should be migrated. at org.apache.kafka.streams.processor.internals.StateManagerUtil.wipeStateStores(StateManagerUtil.java:89) at org.apache.kafka.streams.processor.internals.StreamTask.close(StreamTask.java:450) at org.apache.kafka.streams.processor.internals.StreamTask.closeDirty(StreamTask.java:392) at org.apache.kafka.streams.processor.internals.TaskManager.handleLostAll(TaskManager.java:339) at org.apache.kafka.streams.processor.internals.StreamThread.runLoop(StreamThread.java:766) at org.apache.kafka.streams.processor.internals.StreamThread.run(StreamThread.java:725) [2020-02-29T05:50:03-08:00] (streams-soak-trunk-eos_soak_i-0c3043df582d0379a_streamslog) Caused by: java.nio.file.DirectoryNotEmptyException: /mnt/run/streams/state/stream-soak-test/1_1/KSTREAM-AGGREGATE-STATE-STORE-0000000025/KSTREAM-AGGREGATE-STATE-STORE-0000000025.1582891200000 at sun.nio.fs.UnixFileSystemProvider.implDelete(UnixFileSystemProvider.java:242) at sun.nio.fs.AbstractFileSystemProvider.delete(AbstractFileSystemProvider.java:103) at java.nio.file.Files.delete(Files.java:1126) at org.apache.kafka.common.utils.Utils$2.postVisitDirectory(Utils.java:762) at org.apache.kafka.common.utils.Utils$2.postVisitDirectory(Utils.java:744) at java.nio.file.Files.walkFileTree(Files.java:2688) at java.nio.file.Files.walkFileTree(Files.java:2742) at org.apache.kafka.common.utils.Utils.delete(Utils.java:744) at org.apache.kafka.streams.processor.internals.StateManagerUtil.wipeStateStores(StateManagerUtil.java:86) ... 5 more [2020-02-29T05:50:03-08:00] (streams-soak-trunk-eos_soak_i-0c3043df582d0379a_streamslog) [2020-02-29 13:49:58,305] INFO [stream-soak-test-e6003ae8-1331-4b99-97e3-a3229aaa15fd-StreamThread-3] stream-thread [stream-soak-test-e6003ae8-1331-4b99-97e3-a3229aaa15fd-StreamThread-3] State transition from RUNNING to PENDING_SHUTDOWN (org.apache.kafka.streams.processor.internals.StreamThread) {code} was (Author: bchen225242): Reproduced the exception as: {code:java} [stream-soak-test-776e3009-0514-454c-a473-8964806818c0-StreamThread-1] task [1_1] Failed to wiping state stores for task 1_1 due to {} (org.apache.kafka.streams.processor.internals.StreamTask) [2020-02-28T05:18:02-08:00] (streams-soak-trunk-eos_soak_i-0f3c459ae5907ce97_streamslog) java.nio.file.DirectoryNotEmptyException: /mnt/run/streams/state/stream-soak-test/1_1/KSTREAM-AGGREGATE-STATE-STORE-0000000019/KSTREAM-AGGREGATE-STATE-STORE-0000000019.1582804800000 at sun.nio.fs.UnixFileSystemProvider.implDelete(UnixFileSystemProvider.java:242) at sun.nio.fs.AbstractFileSystemProvider.delete(AbstractFileSystemProvider.java:103) at java.nio.file.Files.delete(Files.java:1126) at org.apache.kafka.common.utils.Utils$2.postVisitDirectory(Utils.java:761) at org.apache.kafka.common.utils.Utils$2.postVisitDirectory(Utils.java:744) at java.nio.file.Files.walkFileTree(Files.java:2688) at java.nio.file.Files.walkFileTree(Files.java:2742) at org.apache.kafka.common.utils.Utils.delete(Utils.java:744) at org.apache.kafka.streams.processor.internals.StateManagerUtil.wipeStateStores(StateManagerUtil.java:84) at org.apache.kafka.streams.processor.internals.StreamTask.close(StreamTask.java:450) at org.apache.kafka.streams.processor.internals.StreamTask.closeDirty(StreamTask.java:392) at org.apache.kafka.streams.processor.internals.TaskManager.handleLostAll(TaskManager.java:337) at org.apache.kafka.streams.processor.internals.StreamThread.runLoop(StreamThread.java:766) at org.apache.kafka.streams.processor.internals.StreamThread.run(StreamThread.java:725) {code} > Failed state store deletion could lead to task file not found > ------------------------------------------------------------- > > Key: KAFKA-9618 > URL: https://issues.apache.org/jira/browse/KAFKA-9618 > Project: Kafka > Issue Type: Bug > Components: streams > Reporter: Boyang Chen > Assignee: Boyang Chen > Priority: Major > > A failed deletion of a stream task directory could later lead to the > impression that the task state is still there, thus causing file not found > exception as the directory was partially deleted. > {code:java} > [2020-02-26T22:08:05-08:00] > (streams-soak-trunk-eos_soak_i-04ebd21fd0e0da9bf_streamslog) [2020-02-27 > 06:08:04,394] WARN > [stream-soak-test-b26adb53-07e2-4013-933a-0f4bcac84c04-StreamThread-2] > stream-thread > [stream-soak-test-b26adb53-07e2-4013-933a-0f4bcac84c04-StreamThread-2] task > [2_2] Failed to wiping state stores for task 2_2 > (org.apache.kafka.streams.processor.internals.StreamTask) > [2020-02-26T22:08:05-08:00] > (streams-soak-trunk-eos_soak_i-04ebd21fd0e0da9bf_streamslog) [2020-02-27 > 06:08:04,394] INFO > [stream-soak-test-b26adb53-07e2-4013-933a-0f4bcac84c04-StreamThread-2] > [Producer > clientId=stream-soak-test-b26adb53-07e2-4013-933a-0f4bcac84c04-StreamThread-2-2_2-producer, > transactionalId=stream-soak-test-2_2] Closing the Kafka producer with > timeoutMillis = 9223372036854775807 ms. > (org.apache.kafka.clients.producer.KafkaProducer) > [2020-02-26T22:08:05-08:00] > (streams-soak-trunk-eos_soak_i-04ebd21fd0e0da9bf_streamslog) [2020-02-27 > 06:08:04,411] ERROR > [stream-soak-test-b26adb53-07e2-4013-933a-0f4bcac84c04-StreamThread-1] > stream-thread > [stream-soak-test-b26adb53-07e2-4013-933a-0f4bcac84c04-StreamThread-1] > Encountered the following exception during processing and the thread is going > to shut down: (org.apache.kafka.streams.processor.internals.StreamThread) > [2020-02-26T22:08:05-08:00] > (streams-soak-trunk-eos_soak_i-04ebd21fd0e0da9bf_streamslog) > org.apache.kafka.streams.errors.ProcessorStateException: Error opening store > KSTREAM-AGGREGATE-STATE-STORE-0000000040 at location > /mnt/run/streams/state/stream-soak-test/2_2/rocksdb/KSTREAM-AGGREGATE-STATE-STORE-0000000040 > at > org.apache.kafka.streams.state.internals.RocksDBTimestampedStore.openRocksDB(RocksDBTimestampedStore.java:87) > at > org.apache.kafka.streams.state.internals.RocksDBStore.openDB(RocksDBStore.java:191) > at > org.apache.kafka.streams.state.internals.RocksDBStore.init(RocksDBStore.java:230) > at > org.apache.kafka.streams.state.internals.WrappedStateStore.init(WrappedStateStore.java:48) > at > org.apache.kafka.streams.state.internals.ChangeLoggingKeyValueBytesStore.init(ChangeLoggingKeyValueBytesStore.java:44) > at > org.apache.kafka.streams.state.internals.WrappedStateStore.init(WrappedStateStore.java:48) > at > org.apache.kafka.streams.state.internals.CachingKeyValueStore.init(CachingKeyValueStore.java:58) > at > org.apache.kafka.streams.state.internals.WrappedStateStore.init(WrappedStateStore.java:48) > {code} -- This message was sent by Atlassian Jira (v8.3.4#803005)