You are viewing a plain text version of this content. The canonical link for it is here.
Posted to jira@kafka.apache.org by "rreddy-22 (via GitHub)" <gi...@apache.org> on 2023/04/07 01:44:35 UTC

[GitHub] [kafka] rreddy-22 opened a new pull request, #13524: KIP-848-Interface changes

rreddy-22 opened a new pull request, #13524:
URL: https://github.com/apache/kafka/pull/13524

   -> Converted Collection<String> to Collection<Uuid> for list of subscriptions
   -> Converted targetPartitions to a map of Uuid (topicIds) to Set<Integers> (partition numbers)
   -> Added a new class TopicIdToPartition 
   


-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163037721


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {
+    private final Uuid topicId;
+    private final Integer partition;
+    private final Optional<List<String>> rackIds;
+
+    public TopicIdToPartition(Uuid topicId, Integer topicPartition, Optional<List<String>> rackIds) {
+        this.topicId = Objects.requireNonNull(topicId, "topicId can not be null");
+        this.partition = Objects.requireNonNull(topicPartition, "topicPartition can not be null");
+        this.rackIds = rackIds;

Review Comment:
   ohh okay got it! thanks!



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164499488


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -29,37 +31,53 @@ public class AssignmentMemberSpec {
     /**
      * The instance ID if provided.
      */
-    final Optional<String> instanceId;
+    private final Optional<String> instanceId;
 
     /**
      * The rack ID if provided.
      */
-    final Optional<String> rackId;
+    private final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    private final Collection<Uuid> subscribedTopics;
 
     /**
-     * The current target partitions of the member.
+     * Partitions assigned for this member keyed by topicId
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> assignedTopicIdPartitions;

Review Comment:
   I thought it would be more clear if it was topicIdpartitions since we're keying by topicId? Should we change it?
   



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] jeffkbkim commented on pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "jeffkbkim (via GitHub)" <gi...@apache.org>.
jeffkbkim commented on PR #13524:
URL: https://github.com/apache/kafka/pull/13524#issuecomment-1505542589

   > We need to add a way to provide the racks. One way would be to augment AssignmentTopicMetadata with them. Another way would be to pass ClusterDescriber interface to PartitionAssignor#assign. The interface would then have a method to get the number of partitions for a topic id and a method to get the racks for a topic id/partition id. What do you think? cc @jeffkbkim
   
   this makes sense to me. it will be used for both assignors


-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1165892001


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -54,6 +60,6 @@ public int hashCode() {
 
     @Override
     public String toString() {
-        return "MemberAssignment(targetPartitions=" + targetPartitions + ')';
+        return "MemberAssignment (Target partitions = " + targetPartitions + ')';

Review Comment:
   fixed thanks



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164535218


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentSpec.java:
##########
@@ -28,12 +28,12 @@ public class AssignmentSpec {
     /**
      * The members keyed by member id.
      */
-    final Map<String, AssignmentMemberSpec> members;
+    private final Map<String, AssignmentMemberSpec> members;
 
     /**
      * The topics' metadata keyed by topic id
      */
-    final Map<Uuid, AssignmentTopicMetadata> topics;
+    private final Map<Uuid, AssignmentTopicMetadata> topics;

Review Comment:
   I didn't think about the full implementation yet and where the rackIds will be passed, figured we could edit it when it came to implementing the algorithm. The topicIdToPartition data structure could be populated using this so maybe we can add a field for a map of partition to its rackIds



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164542031


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {
+    private final Uuid topicId;
+    private final Integer partition;
+    private final Optional<List<String>> rackIds;
+
+    public TopicIdToPartition(Uuid topicId, Integer topicPartition, Optional<List<String>> rackIds) {
+        this.topicId = Objects.requireNonNull(topicId, "topicId cannot be null");
+        this.partition = Objects.requireNonNull(topicPartition, "topicPartition cannot be null");
+        this.rackIds = Objects.requireNonNull(rackIds, "rackId cannot be null");
+    }
+
+    /**
+     * @return Universally unique id representing this topic partition.
+     */
+    public Uuid topicId() {
+        return topicId;
+    }
+
+    /**
+     * @return the partition number.
+     */
+    public int partition() {
+        return partition;
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o) {
+            return true;
+        }
+        if (o == null || getClass() != o.getClass()) {
+            return false;
+        }
+        TopicIdToPartition that = (TopicIdToPartition) o;
+        return topicId.equals(that.topicId) &&
+                partition.equals(that.partition);
+    }
+
+    @Override
+    public int hashCode() {
+        final int prime = 31;

Review Comment:
   Hey, yeah we wanted a data structure that only had a topicId to partition number mapping for each partition. The existing topicIdPartition class has topicNames as well. I didn't know how else to name it uniquely xD



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] jeffkbkim commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "jeffkbkim (via GitHub)" <gi...@apache.org>.
jeffkbkim commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1161928058


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {
+    private final Uuid topicId;
+    private final Integer partition;
+    private final Optional<List<String>> rackIds;
+
+    public TopicIdToPartition(Uuid topicId, Integer topicPartition, Optional<List<String>> rackIds) {
+        this.topicId = Objects.requireNonNull(topicId, "topicId can not be null");
+        this.partition = Objects.requireNonNull(topicPartition, "topicPartition can not be null");
+        this.rackIds = rackIds;

Review Comment:
   this should also be not null. If rackIds is empty, we should pass in Optional.empty()



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac merged pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac merged PR #13524:
URL: https://github.com/apache/kafka/pull/13524


-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163035038


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   I thought we wanted all the changes in one place including the topicIdPartition structure 



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163038733


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   Right. I would like to have all the changes related to the interface in one PR. However, it must be part of the interface... At the moment, `TopicIdToPartition` is not used. Or, am I missing something?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   Right. I would like to have all the changes related to the interface in one PR. However, it must be part of the interface... At the moment, `TopicIdToPartition` is not used. Or, am I missing something?



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1165679441


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentTopicMetadata.java:
##########
@@ -25,12 +25,12 @@ public class AssignmentTopicMetadata {
     /**
      * The topic name.
      */
-    final String topicName;
+    private final String topicName;

Review Comment:
   okay 



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1160999928


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -16,25 +16,28 @@
  */
 package org.apache.kafka.coordinator.group.assignor;
 
-import org.apache.kafka.common.TopicPartition;
+import org.apache.kafka.common.Uuid;
 
-import java.util.Collection;
+import java.util.Map;
 import java.util.Objects;
+import java.util.Set;
 
 /**
  * The partition assignment for a consumer group member.
  */
 public class MemberAssignment {
     /**
-     * The target partitions assigned to this member.
+     * The target partitions assigned to this member grouped by topicId.
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> topicIdPartitionsMap;

Review Comment:
   I was wondering if it'll be confused with the topicIdToPartition class



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1166456295


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   @Hangleton This is class is not in the PR any more.



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1160999728


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   Used it in the uniform assignor 



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163354734


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -37,29 +39,30 @@ public class AssignmentMemberSpec {
     final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.

Review Comment:
   consumers usually subscribe using topic names so I assumed topics would mean topic name, to make sure we know its the topic Id generated by the coordinator I said topicIds of the topics that the member is subscribed to



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164536324


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentTopicMetadata.java:
##########
@@ -25,12 +25,12 @@ public class AssignmentTopicMetadata {
     /**
      * The topic name.
      */
-    final String topicName;
+    private final String topicName;

Review Comment:
   I think there's no harm in keeping it just in case we want to know what topic name is associated with what topicId wdyt?



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1162846084


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -37,29 +39,30 @@ public class AssignmentMemberSpec {
     final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    final Collection<Uuid> subscribedTopics;
 
     /**
-     * The current target partitions of the member.
+     * Partitions assigned for this member grouped by topicId
      */
-    final Collection<TopicPartition> targetPartitions;
+    final Map<Uuid, Set<Integer>> currentAssignmentTopicIdPartitions;

Review Comment:
   nit: `assignment` or `assignedPartitions`? Using `current` is not really appropriate here because this field is filled in with the last computed target assignment. 



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -16,25 +16,28 @@
  */
 package org.apache.kafka.coordinator.group.assignor;
 
-import org.apache.kafka.common.TopicPartition;
+import org.apache.kafka.common.Uuid;
 
-import java.util.Collection;
+import java.util.Map;
 import java.util.Objects;
+import java.util.Set;
 
 /**
  * The partition assignment for a consumer group member.
  */
 public class MemberAssignment {
     /**
-     * The target partitions assigned to this member.
+     * The target partitions assigned to this member grouped by topicId.
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> topicIdPartitionsMap;

Review Comment:
   nit: `assignedPartitions` or `assignment`?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -89,7 +92,7 @@ public String toString() {
         return "AssignmentMemberSpec(instanceId=" + instanceId +
             ", rackId=" + rackId +
             ", subscribedTopics=" + subscribedTopics +
-            ", targetPartitions=" + targetPartitions +
+            ", targetPartitions=" + currentAssignmentTopicIdPartitions +

Review Comment:
   nit: We need to update the string as well.



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -37,29 +39,30 @@ public class AssignmentMemberSpec {
     final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.

Review Comment:
   nit: `The topic ids that the member is subscribed to.`?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -44,16 +47,16 @@ public boolean equals(Object o) {
 
         MemberAssignment that = (MemberAssignment) o;
 
-        return targetPartitions.equals(that.targetPartitions);
+        return topicIdPartitionsMap.equals(that.topicIdPartitionsMap);
     }
 
     @Override
     public int hashCode() {
-        return targetPartitions.hashCode();
+        return topicIdPartitionsMap.hashCode();
     }
 
     @Override
     public String toString() {
-        return "MemberAssignment(targetPartitions=" + targetPartitions + ')';
+        return "MemberAssignment ( Assignment per topic Id = " + topicIdPartitionsMap + ')';

Review Comment:
   nit: We always use the name of the variable as string, followed by `=`, and the value. In this case, `topicIdPartitionsMap= + topicIdPartitionsMap`. Moreover, we don't put spaces before/after the `(`.



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   Is it used in the interface? If not, let's remove it from this PR.



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] jolshan commented on pull request #13524: KIP-848-Interface changes

Posted by "jolshan (via GitHub)" <gi...@apache.org>.
jolshan commented on PR #13524:
URL: https://github.com/apache/kafka/pull/13524#issuecomment-1500714508

   @rreddy-22 is there a jira for this change? If so can we use it in the title like 
   KAFKA-XYZ: Name of the JIRA?


-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163355154


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -37,29 +39,30 @@ public class AssignmentMemberSpec {
     final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    final Collection<Uuid> subscribedTopics;
 
     /**
-     * The current target partitions of the member.
+     * Partitions assigned for this member grouped by topicId
      */
-    final Collection<TopicPartition> targetPartitions;
+    final Map<Uuid, Set<Integer>> currentAssignmentTopicIdPartitions;

Review Comment:
   makes sense, I was also thinking of changing it, lemme change it here



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164530365


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -16,25 +16,28 @@
  */
 package org.apache.kafka.coordinator.group.assignor;
 
-import org.apache.kafka.common.TopicPartition;
+import org.apache.kafka.common.Uuid;
 
-import java.util.Collection;
+import java.util.Map;
 import java.util.Objects;
+import java.util.Set;
 
 /**
  * The partition assignment for a consumer group member.
  */
 public class MemberAssignment {
     /**
-     * The target partitions assigned to this member.
+     * The target partitions assigned to this member keyed by topicId.
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> assignedTopicIdPartitions;

Review Comment:
   sounds good



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163998694


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -37,29 +39,30 @@ public class AssignmentMemberSpec {
     final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.

Review Comment:
   Isn't `topic ids` precise enough?



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: KIP-848-Interface changes

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1160578628


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   Is this used anywhere? I may have missed it.



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1165149138


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -54,6 +60,6 @@ public int hashCode() {
 
     @Override
     public String toString() {
-        return "MemberAssignment(targetPartitions=" + targetPartitions + ')';
+        return "MemberAssignment (Target partitions = " + targetPartitions + ')';

Review Comment:
   nit: The format does not follow our convention whereas the former code did. Could we revert this line back?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentTopicMetadata.java:
##########
@@ -25,12 +25,12 @@ public class AssignmentTopicMetadata {
     /**
      * The topic name.
      */
-    final String topicName;
+    private final String topicName;

Review Comment:
   I would remove it because I don't really see the value of having it. All the interface works with topic ids now.



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] dajac commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "dajac (via GitHub)" <gi...@apache.org>.
dajac commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163998872


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -29,37 +31,53 @@ public class AssignmentMemberSpec {
     /**
      * The instance ID if provided.
      */
-    final Optional<String> instanceId;
+    private final Optional<String> instanceId;
 
     /**
      * The rack ID if provided.
      */
-    final Optional<String> rackId;
+    private final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    private final Collection<Uuid> subscribedTopics;

Review Comment:
   nit: `subscribedTopicIds`



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -29,37 +31,53 @@ public class AssignmentMemberSpec {
     /**
      * The instance ID if provided.
      */
-    final Optional<String> instanceId;
+    private final Optional<String> instanceId;
 
     /**
      * The rack ID if provided.
      */
-    final Optional<String> rackId;
+    private final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    private final Collection<Uuid> subscribedTopics;
 
     /**
-     * The current target partitions of the member.
+     * Partitions assigned for this member keyed by topicId
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> assignedTopicIdPartitions;

Review Comment:
   nit: Sorry to repeat myself but how about `assignedPartitions`?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -29,37 +31,53 @@ public class AssignmentMemberSpec {
     /**
      * The instance ID if provided.
      */
-    final Optional<String> instanceId;
+    private final Optional<String> instanceId;
 
     /**
      * The rack ID if provided.
      */
-    final Optional<String> rackId;
+    private final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    private final Collection<Uuid> subscribedTopics;
 
     /**
-     * The current target partitions of the member.
+     * Partitions assigned for this member keyed by topicId

Review Comment:
   nit: Add missing `.` at the end.



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*

Review Comment:
   What do we do with this file?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentTopicMetadata.java:
##########
@@ -25,12 +25,12 @@ public class AssignmentTopicMetadata {
     /**
      * The topic name.
      */
-    final String topicName;
+    private final String topicName;

Review Comment:
   We should probably remove the `topicName` as we only use `topicId`.



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -44,16 +47,16 @@ public boolean equals(Object o) {
 
         MemberAssignment that = (MemberAssignment) o;
 
-        return targetPartitions.equals(that.targetPartitions);
+        return assignedTopicIdPartitions.equals(that.assignedTopicIdPartitions);
     }
 
     @Override
     public int hashCode() {
-        return targetPartitions.hashCode();
+        return assignedTopicIdPartitions.hashCode();
     }
 
     @Override
     public String toString() {
-        return "MemberAssignment(targetPartitions=" + targetPartitions + ')';
+        return "MemberAssignment (Assignment per topic Id = " + assignedTopicIdPartitions + ')';

Review Comment:
   nit: `"MemberAssignment(assignedTopicIdPartitions=" + assignedTopicIdPartitions + ')';`



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentSpec.java:
##########
@@ -28,12 +28,12 @@ public class AssignmentSpec {
     /**
      * The members keyed by member id.
      */
-    final Map<String, AssignmentMemberSpec> members;
+    private final Map<String, AssignmentMemberSpec> members;
 
     /**
      * The topics' metadata keyed by topic id
      */
-    final Map<Uuid, AssignmentTopicMetadata> topics;
+    private final Map<Uuid, AssignmentTopicMetadata> topics;

Review Comment:
   Did you mean to add the partitions with their rack ids to `AssignmentTopicMetadata`? What's your plan?



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/MemberAssignment.java:
##########
@@ -16,25 +16,28 @@
  */
 package org.apache.kafka.coordinator.group.assignor;
 
-import org.apache.kafka.common.TopicPartition;
+import org.apache.kafka.common.Uuid;
 
-import java.util.Collection;
+import java.util.Map;
 import java.util.Objects;
+import java.util.Set;
 
 /**
  * The partition assignment for a consumer group member.
  */
 public class MemberAssignment {
     /**
-     * The target partitions assigned to this member.
+     * The target partitions assigned to this member keyed by topicId.
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> assignedTopicIdPartitions;

Review Comment:
   nit: `targetPartitions` is actually good here or `assignedPartitions` again.



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -29,37 +31,53 @@ public class AssignmentMemberSpec {
     /**
      * The instance ID if provided.
      */
-    final Optional<String> instanceId;
+    private final Optional<String> instanceId;
 
     /**
      * The rack ID if provided.
      */
-    final Optional<String> rackId;
+    private final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.
      */
-    final Collection<String> subscribedTopics;
+    private final Collection<Uuid> subscribedTopics;
 
     /**
-     * The current target partitions of the member.
+     * Partitions assigned for this member keyed by topicId
      */
-    final Collection<TopicPartition> targetPartitions;
+    private final Map<Uuid, Set<Integer>> assignedTopicIdPartitions;
+
+    public Optional<String> instanceId() {

Review Comment:
   nit: Let's add javadoc to all those accessors. It can by as simple as:
   ```
   /**
    * @ return The instance id as an Optional.
    */
   ```



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164538097


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*

Review Comment:
   we can remove it here if its not the right place for it and then couple it with the assignor PRs where its used



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1165681079


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {
+    private final Uuid topicId;
+    private final Integer partition;
+    private final Optional<List<String>> rackIds;
+
+    public TopicIdToPartition(Uuid topicId, Integer topicPartition, Optional<List<String>> rackIds) {
+        this.topicId = Objects.requireNonNull(topicId, "topicId cannot be null");
+        this.partition = Objects.requireNonNull(topicPartition, "topicPartition cannot be null");
+        this.rackIds = Objects.requireNonNull(rackIds, "rackId cannot be null");
+    }
+
+    /**
+     * @return Universally unique id representing this topic partition.
+     */
+    public Uuid topicId() {
+        return topicId;
+    }
+
+    /**
+     * @return the partition number.
+     */
+    public int partition() {
+        return partition;
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o) {
+            return true;
+        }
+        if (o == null || getClass() != o.getClass()) {
+            return false;
+        }
+        TopicIdToPartition that = (TopicIdToPartition) o;
+        return topicId.equals(that.topicId) &&
+                partition.equals(that.partition);
+    }
+
+    @Override
+    public int hashCode() {
+        final int prime = 31;

Review Comment:
   that sounds good! We'll name it that when we add the file! Thanks!



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1163356477


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -89,7 +92,7 @@ public String toString() {
         return "AssignmentMemberSpec(instanceId=" + instanceId +
             ", rackId=" + rackId +
             ", subscribedTopics=" + subscribedTopics +
-            ", targetPartitions=" + targetPartitions +
+            ", targetPartitions=" + currentAssignmentTopicIdPartitions +

Review Comment:
   done mb



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] Hangleton commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "Hangleton (via GitHub)" <gi...@apache.org>.
Hangleton commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1164088051


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {
+    private final Uuid topicId;
+    private final Integer partition;
+    private final Optional<List<String>> rackIds;
+
+    public TopicIdToPartition(Uuid topicId, Integer topicPartition, Optional<List<String>> rackIds) {
+        this.topicId = Objects.requireNonNull(topicId, "topicId cannot be null");
+        this.partition = Objects.requireNonNull(topicPartition, "topicPartition cannot be null");
+        this.rackIds = Objects.requireNonNull(rackIds, "rackId cannot be null");
+    }
+
+    /**
+     * @return Universally unique id representing this topic partition.
+     */
+    public Uuid topicId() {
+        return topicId;
+    }
+
+    /**
+     * @return the partition number.
+     */
+    public int partition() {
+        return partition;
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o) {
+            return true;
+        }
+        if (o == null || getClass() != o.getClass()) {
+            return false;
+        }
+        TopicIdToPartition that = (TopicIdToPartition) o;
+        return topicId.equals(that.topicId) &&
+                partition.equals(that.partition);
+    }
+
+    @Override
+    public int hashCode() {
+        final int prime = 31;

Review Comment:
   nit: `Objects.hashCode()` could be of use here.



##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,50 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {

Review Comment:
   Nit: sorry I keep coming after names, but what does topic id to partition mean? Is this class a `TopicIdPartition` w/o the topic name and augmented by a list of rack ids? If so, the differentiating characteristic of this class seems to be the location of the partition. Should that be reflected in the class name?



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on PR #13524:
URL: https://github.com/apache/kafka/pull/13524#issuecomment-1505791579

   > 
   
   
   
   > 
   
   I like the idea of augmenting AssignmentTopicMetadata with rackIds since this comes under metadata and it'll be passed to all the assignors 


-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] Hangleton commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "Hangleton (via GitHub)" <gi...@apache.org>.
Hangleton commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1165311719


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/common/TopicIdToPartition.java:
##########
@@ -0,0 +1,76 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements. See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You under the Apache License, Version 2.0
+ * (the "License"); you may not use this file except in compliance with
+ * the License. You may obtain a copy of the License at
+ *
+ *    http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.kafka.coordinator.group.common;
+
+import org.apache.kafka.common.Uuid;
+
+import java.util.List;
+import java.util.Objects;
+import java.util.Optional;
+
+public class TopicIdToPartition {
+    private final Uuid topicId;
+    private final Integer partition;
+    private final Optional<List<String>> rackIds;
+
+    public TopicIdToPartition(Uuid topicId, Integer topicPartition, Optional<List<String>> rackIds) {
+        this.topicId = Objects.requireNonNull(topicId, "topicId cannot be null");
+        this.partition = Objects.requireNonNull(topicPartition, "topicPartition cannot be null");
+        this.rackIds = Objects.requireNonNull(rackIds, "rackId cannot be null");
+    }
+
+    /**
+     * @return Universally unique id representing this topic partition.
+     */
+    public Uuid topicId() {
+        return topicId;
+    }
+
+    /**
+     * @return the partition number.
+     */
+    public int partition() {
+        return partition;
+    }
+
+    @Override
+    public boolean equals(Object o) {
+        if (this == o) {
+            return true;
+        }
+        if (o == null || getClass() != o.getClass()) {
+            return false;
+        }
+        TopicIdToPartition that = (TopicIdToPartition) o;
+        return topicId.equals(that.topicId) &&
+                partition.equals(that.partition);
+    }
+
+    @Override
+    public int hashCode() {
+        final int prime = 31;

Review Comment:
   I see, thanks for explaining. Rack-aware topic id partition? 



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [kafka] rreddy-22 commented on a diff in pull request #13524: MINOR: Refine `PartitionAssignor` server-side interface

Posted by "rreddy-22 (via GitHub)" <gi...@apache.org>.
rreddy-22 commented on code in PR #13524:
URL: https://github.com/apache/kafka/pull/13524#discussion_r1165678602


##########
group-coordinator/src/main/java/org/apache/kafka/coordinator/group/assignor/AssignmentMemberSpec.java:
##########
@@ -37,29 +39,30 @@ public class AssignmentMemberSpec {
     final Optional<String> rackId;
 
     /**
-     * The topics that the member is subscribed to.
+     * The topicIds of topics that the member is subscribed to.

Review Comment:
   yep changed it 



-- 
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-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org