junrao commented on code in PR #12005:
URL: https://github.com/apache/kafka/pull/12005#discussion_r865419210


##########
core/src/main/scala/kafka/server/LeaderEndPoint.scala:
##########
@@ -0,0 +1,106 @@
+/*
+ * 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 kafka.server
+
+import kafka.server.AbstractFetcherThread.{ReplicaFetch, ResultWithPartitions}
+import org.apache.kafka.common.TopicPartition
+import org.apache.kafka.common.requests.FetchRequest
+import 
org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset
+import org.apache.kafka.common.message.{FetchResponseData, 
OffsetForLeaderEpochRequestData}
+
+import scala.collection.Map
+
+/**
+ * The LeaderEndPoint acts as an abstraction which serves all fetches from the 
leader
+ * for the fetcher threads.
+ */
+trait LeaderEndPoint {
+
+  type FetchData = FetchResponseData.PartitionData
+  type EpochData = OffsetForLeaderEpochRequestData.OffsetForLeaderPartition
+
+  /**
+   * A boolean specifying if truncation when fetching from the leader is 
supported
+   */
+  def isTruncationOnFetchSupported: Boolean
+
+  /**
+   * Initiate closing access to fetches from leader.
+   */
+  def initiateClose(): Unit
+
+  /**
+   * Closes access to fetches from leader.
+   */
+  def close(): Unit
+
+  /**
+   * Given a fetchRequest, carries out the expected request and returns
+   * the results from fetching from the leader.
+   *
+   * @param fetchRequest The fetch request we want to carry out
+   *
+   * @return A map of topic partition -> fetch data
+   */
+  def fetch(fetchRequest: FetchRequest.Builder): Map[TopicPartition, FetchData]
+
+  /**
+   * Fetches the log start offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from
+   *
+   * @return A long representing the earliest offset in the leader's topic 
partition.
+   */
+  def fetchEarliestOffset(topicPartition: TopicPartition, currentLeaderEpoch: 
Int): Long
+
+  /**
+   * Fetches the log end offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from

Review Comment:
   An int representing the current leader epoch that we want to fetch from => 
current leader epoch of the requestor



##########
core/src/main/scala/kafka/server/LeaderEndPoint.scala:
##########
@@ -0,0 +1,106 @@
+/*
+ * 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 kafka.server
+
+import kafka.server.AbstractFetcherThread.{ReplicaFetch, ResultWithPartitions}
+import org.apache.kafka.common.TopicPartition
+import org.apache.kafka.common.requests.FetchRequest
+import 
org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset
+import org.apache.kafka.common.message.{FetchResponseData, 
OffsetForLeaderEpochRequestData}
+
+import scala.collection.Map
+
+/**
+ * The LeaderEndPoint acts as an abstraction which serves all fetches from the 
leader
+ * for the fetcher threads.
+ */
+trait LeaderEndPoint {
+
+  type FetchData = FetchResponseData.PartitionData
+  type EpochData = OffsetForLeaderEpochRequestData.OffsetForLeaderPartition
+
+  /**
+   * A boolean specifying if truncation when fetching from the leader is 
supported
+   */
+  def isTruncationOnFetchSupported: Boolean
+
+  /**
+   * Initiate closing access to fetches from leader.
+   */
+  def initiateClose(): Unit
+
+  /**
+   * Closes access to fetches from leader.
+   */
+  def close(): Unit
+
+  /**
+   * Given a fetchRequest, carries out the expected request and returns
+   * the results from fetching from the leader.
+   *
+   * @param fetchRequest The fetch request we want to carry out
+   *
+   * @return A map of topic partition -> fetch data
+   */
+  def fetch(fetchRequest: FetchRequest.Builder): Map[TopicPartition, FetchData]
+
+  /**
+   * Fetches the log start offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from

Review Comment:
   An int representing the current leader epoch that we want to fetch from => 
current leader epoch of the requestor



##########
core/src/main/scala/kafka/server/LeaderEndPoint.scala:
##########
@@ -0,0 +1,106 @@
+/*
+ * 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 kafka.server
+
+import kafka.server.AbstractFetcherThread.{ReplicaFetch, ResultWithPartitions}
+import org.apache.kafka.common.TopicPartition
+import org.apache.kafka.common.requests.FetchRequest
+import 
org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset
+import org.apache.kafka.common.message.{FetchResponseData, 
OffsetForLeaderEpochRequestData}
+
+import scala.collection.Map
+
+/**
+ * The LeaderEndPoint acts as an abstraction which serves all fetches from the 
leader
+ * for the fetcher threads.
+ */
+trait LeaderEndPoint {
+
+  type FetchData = FetchResponseData.PartitionData
+  type EpochData = OffsetForLeaderEpochRequestData.OffsetForLeaderPartition
+
+  /**
+   * A boolean specifying if truncation when fetching from the leader is 
supported
+   */
+  def isTruncationOnFetchSupported: Boolean
+
+  /**
+   * Initiate closing access to fetches from leader.
+   */
+  def initiateClose(): Unit
+
+  /**
+   * Closes access to fetches from leader.
+   */
+  def close(): Unit
+
+  /**
+   * Given a fetchRequest, carries out the expected request and returns
+   * the results from fetching from the leader.
+   *
+   * @param fetchRequest The fetch request we want to carry out
+   *
+   * @return A map of topic partition -> fetch data
+   */
+  def fetch(fetchRequest: FetchRequest.Builder): Map[TopicPartition, FetchData]
+
+  /**
+   * Fetches the log start offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from
+   *
+   * @return A long representing the earliest offset in the leader's topic 
partition.
+   */
+  def fetchEarliestOffset(topicPartition: TopicPartition, currentLeaderEpoch: 
Int): Long
+
+  /**
+   * Fetches the log end offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from
+   *
+   * @return A long representing the latest offset in the leader's topic 
partition.
+   */
+  def fetchLatestOffset(topicPartition: TopicPartition, currentLeaderEpoch: 
Int): Long
+
+  /**
+   * Fetches offset for leader epoch from the leader for each given topic 
partition
+   *
+   * @param partitions A map of topic partition -> leader epoch of the future 
replica
+   *
+   * @return A map of topic partition -> end offset for a requested leader 
epoch
+   */
+  def fetchEpochEndOffsets(partitions: Map[TopicPartition, EpochData]): 
Map[TopicPartition, EpochEndOffset]
+
+  /**
+   * Builds a fetch request, given a partition map.
+   *
+   * @param partitionMap A map of topic partitions to their respective 
partition fetch state
+   *
+   * @return A ResultWithPartitions, used to create the fetchRequest for fetch.
+   */
+  def buildFetch(partitionMap: Map[TopicPartition, PartitionFetchState]): 
ResultWithPartitions[Option[ReplicaFetch]]
+
+  /**
+   * Returns a string representation of the LeaderEndPoint.
+   */
+  override def toString: String = super.toString

Review Comment:
   Do we need this?



##########
core/src/main/scala/kafka/server/AbstractFetcherThread.scala:
##########
@@ -51,6 +51,7 @@ import scala.math._
  */
 abstract class AbstractFetcherThread(name: String,
                                      clientId: String,
+                                     val leader: LeaderEndPoint,

Review Comment:
   It's a bit weird to pass in both LeaderEndPoint and BrokerEndPoint since the 
former includes the latter. We could address that in a separate PR if easier.



##########
core/src/main/scala/kafka/server/LeaderEndPoint.scala:
##########
@@ -0,0 +1,106 @@
+/*
+ * 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 kafka.server
+
+import kafka.server.AbstractFetcherThread.{ReplicaFetch, ResultWithPartitions}
+import org.apache.kafka.common.TopicPartition
+import org.apache.kafka.common.requests.FetchRequest
+import 
org.apache.kafka.common.message.OffsetForLeaderEpochResponseData.EpochEndOffset
+import org.apache.kafka.common.message.{FetchResponseData, 
OffsetForLeaderEpochRequestData}
+
+import scala.collection.Map
+
+/**
+ * The LeaderEndPoint acts as an abstraction which serves all fetches from the 
leader
+ * for the fetcher threads.
+ */
+trait LeaderEndPoint {
+
+  type FetchData = FetchResponseData.PartitionData
+  type EpochData = OffsetForLeaderEpochRequestData.OffsetForLeaderPartition
+
+  /**
+   * A boolean specifying if truncation when fetching from the leader is 
supported
+   */
+  def isTruncationOnFetchSupported: Boolean
+
+  /**
+   * Initiate closing access to fetches from leader.
+   */
+  def initiateClose(): Unit
+
+  /**
+   * Closes access to fetches from leader.
+   */
+  def close(): Unit
+
+  /**
+   * Given a fetchRequest, carries out the expected request and returns
+   * the results from fetching from the leader.
+   *
+   * @param fetchRequest The fetch request we want to carry out
+   *
+   * @return A map of topic partition -> fetch data
+   */
+  def fetch(fetchRequest: FetchRequest.Builder): Map[TopicPartition, FetchData]
+
+  /**
+   * Fetches the log start offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from
+   *
+   * @return A long representing the earliest offset in the leader's topic 
partition.
+   */
+  def fetchEarliestOffset(topicPartition: TopicPartition, currentLeaderEpoch: 
Int): Long
+
+  /**
+   * Fetches the log end offset of the given topic partition, at a specific
+   * leader epoch, from the leader.
+   *
+   * @param topicPartition The topic partition that we want to fetch from
+   * @param currentLeaderEpoch An int representing the current leader epoch 
that we want to fetch from
+   *
+   * @return A long representing the latest offset in the leader's topic 
partition.
+   */
+  def fetchLatestOffset(topicPartition: TopicPartition, currentLeaderEpoch: 
Int): Long
+
+  /**
+   * Fetches offset for leader epoch from the leader for each given topic 
partition
+   *
+   * @param partitions A map of topic partition -> leader epoch of the future 
replica

Review Comment:
   We probably want to remove "future" since this is a generic trait.



-- 
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

Reply via email to