JimmyWang6 commented on code in PR #19820: URL: https://github.com/apache/kafka/pull/19820#discussion_r2166261902
########## tools/src/main/java/org/apache/kafka/tools/consumer/group/ShareGroupCommand.java: ########## @@ -366,6 +378,71 @@ Entry<Throwable, Map<String, Throwable>> sendDeleteShareGroupOffsetsRequest(Stri return new SimpleImmutableEntry<>(topLevelException, topicLevelResult); } + void resetOffsets() throws ExecutionException, InterruptedException { + String groupId = opts.options.valueOf(opts.groupOpt); + Map<TopicPartition, OffsetAndMetadata> offsetsToReset = prepareOffsetsToReset(groupId); + boolean dryRun = opts.options.has(opts.dryRunOpt) || !opts.options.has(opts.executeOpt); + if (!dryRun) { + adminClient.alterShareGroupOffsets(groupId, + offsetsToReset.entrySet().stream() + .collect(Collectors.toMap( + Entry::getKey, + entry -> entry.getValue().offset() + )) + ).all().get(); + } + Set<SharePartitionOffsetInformation> partitionOffsets = mapOffsetsToSharePartitionInformation(groupId, offsetsToReset); + Map<String, ShareGroupDescription> shareGroups = describeShareGroups(Collections.singleton(groupId)); + TreeMap<String, Entry<ShareGroupDescription, Collection<SharePartitionOffsetInformation>>> groupOffsets = new TreeMap<>(); + shareGroups.forEach((id, description) -> groupOffsets.put(id, new SimpleImmutableEntry<>(description, partitionOffsets))); + printOffsets(groupOffsets, opts.options.has(opts.verboseOpt)); + } + + private Map<TopicPartition, OffsetAndMetadata> prepareOffsetsToReset(String groupId) throws ExecutionException, InterruptedException { + Map<String, ListShareGroupOffsetsSpec> groupSpecs = Map.of(groupId, new ListShareGroupOffsetsSpec()); + Map<TopicPartition, OffsetAndMetadata> offsetsByTopicPartitions = adminClient.listShareGroupOffsets(groupSpecs).all().get().get(groupId); + + if (opts.options.has(opts.topicOpt)) { + Set<String> topics = Set.copyOf(opts.options.valuesOf(opts.topicOpt)); + Set<String> existsTopics = offsetsByTopicPartitions.keySet().stream() + .map(TopicPartition::topic) + .collect(Collectors.toSet()); + if (!existsTopics.containsAll(topics)) { + CommandLineUtils + .printUsageAndExit(opts.parser, String.format("Some topics %s do not exist in share group '%s'.", + topics.stream().filter(topic -> !existsTopics.contains(topic)).collect(Collectors.joining(", ")), groupId)); + } + + offsetsByTopicPartitions = offsetsByTopicPartitions.entrySet().stream() + .filter(entry -> { + TopicPartition topicPartition = entry.getKey(); + return topics.contains(topicPartition.topic()); + }) Review Comment: Updated. Thanks for your review. ########## group-coordinator/src/main/java/org/apache/kafka/coordinator/group/GroupCoordinatorService.java: ########## @@ -682,15 +682,48 @@ CompletableFuture<AlterShareGroupOffsetsResponseData> persisterInitialize( handlePersisterInitializeResponse(request.groupTopicPartitionData().groupId(), result, new ShareGroupHeartbeatResponseData()); return response; } else { - //TODO build new AlterShareGroupOffsetsResponseData for error response - return response; + return buildErrorResponse(response, result); } } else { return buildErrorResponse(request, response, exp); } }); } + + private AlterShareGroupOffsetsResponseData buildErrorResponse(AlterShareGroupOffsetsResponseData response, InitializeShareGroupStateResult result) { + AlterShareGroupOffsetsResponseData data = new AlterShareGroupOffsetsResponseData(); + data.setResponses( + new AlterShareGroupOffsetsResponseData.AlterShareGroupOffsetsResponseTopicCollection(response.responses().stream() + .map(topic -> { + AlterShareGroupOffsetsResponseData.AlterShareGroupOffsetsResponseTopic topicData = new AlterShareGroupOffsetsResponseData.AlterShareGroupOffsetsResponseTopic() + .setTopicName(topic.topicName()); + Map<Uuid, Map<Integer, PartitionErrorData>> topicPartitionErrorsMap = result.getErrors(); + topic.partitions().forEach(partition -> { + if (partition.errorCode() != Errors.NONE.code()) { + topicData.partitions().add(partition); + return; + } + AlterShareGroupOffsetsResponseData.AlterShareGroupOffsetsResponsePartition partitionData; + PartitionErrorData error = topicPartitionErrorsMap.get(topic.topicId()).get(partition.partitionIndex()); + if (error == null) { + partitionData = new AlterShareGroupOffsetsResponseData.AlterShareGroupOffsetsResponsePartition() + .setPartitionIndex(partition.partitionIndex()) + .setErrorCode(Errors.NONE.code()); + } else { + partitionData = new AlterShareGroupOffsetsResponseData.AlterShareGroupOffsetsResponsePartition() + .setPartitionIndex(partition.partitionIndex()) + .setErrorCode(error.errorCode()) + .setErrorMessage(error.errorMessage()); + } + topicData.partitions().add(partitionData); + }); + return topicData; + }) + .iterator())); + return data; + Review Comment: Fixed. -- 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. To unsubscribe, e-mail: jira-unsubscr...@kafka.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org