jeqo commented on a change in pull request #10042: URL: https://github.com/apache/kafka/pull/10042#discussion_r573239173
########## File path: core/src/main/scala/kafka/tools/StreamsResetter.java ########## @@ -503,7 +506,16 @@ private void resetToDatetime(final Consumer<byte[], byte[]> client, final Map<TopicPartition, OffsetAndTimestamp> topicPartitionsAndOffset = client.offsetsForTimes(topicPartitionsAndTimes); for (final TopicPartition topicPartition : inputTopicPartitions) { - client.seek(topicPartition, topicPartitionsAndOffset.get(topicPartition).offset()); + final Optional<Long> partitionOffset = Optional.ofNullable(topicPartitionsAndOffset.get(topicPartition)) + .map(OffsetAndTimestamp::offset) + .filter(offset -> offset != ListOffsetsResponse.UNKNOWN_OFFSET); + if (partitionOffset.isPresent()) { + client.seek(topicPartition, partitionOffset.get()); + } else { + client.seekToEnd(Collections.singletonList(topicPartition)); + System.out.println("Partition " + topicPartition.partition() + " from topic " + topicPartition.topic() + + " is empty, without a committed offset. Falling back to latest offset."); Review comment: nit: ```suggestion " is empty, without a committed record. Falling back to latest known offset."); ``` ########## File path: streams/src/test/java/org/apache/kafka/streams/tools/StreamsResetterTest.java ########## @@ -247,6 +268,26 @@ public void shouldDetermineInternalTopicBasedOnTopicName1() { assertTrue(streamsResetter.matchesInternalTopicFormat("appId-KTABLE-FK-JOIN-SUBSCRIPTION-REGISTRATION-12323232-topic")); } + @Test + public void emptyPartitionsAreCorrectlyHandledWhenResettingByDateAndTime() { + final MockConsumer<byte[], byte[]> emptyConsumer = new EmptyPartitionConsumer<>(OffsetResetStrategy.EARLIEST); + final Duration yesterday = Duration.ofDays(1); Review comment: nit: could we move this initialization closer to its usage in L285? or just `.minus(Duration.ofDays(1))` could be simpler. ---------------------------------------------------------------- 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