> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/cluster/Partition.scala, line 245
> > <https://reviews.apache.org/r/24676/diff/11/?file=724366#file724366line245>
> >
> >     Maybe use this:
> >     "Recorded replica %d log end offset (LEO)..."
> >     
> >     Also, instead of an explicit [%s,%d] format specifier I think we should 
> > start doing the following:
> >     
> >     "%s".format(TopicAndPartition(topic, partition))
> >     
> >     That way we ensure a canonical toString for topic/partition pairs and 
> > can change it in one place in the future.
> >     
> >     There are some places where we don't log with this agreed-upon format 
> > and it is a bit annoying, so going forward I think we should use the above. 
> > Can we add it to the logging improvements wiki?

Updated the logging wiki. We can refer people to it when we make logging format 
comments moving forward.


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/cluster/Partition.scala, line 259
> > <https://reviews.apache.org/r/24676/diff/11/?file=724366#file724366line259>
> >
> >     Since we still may update the HW shall we rename this to 
> > maybeUpdateHWAndExpandIsr

The reason I changed its name is that the original name is a bit misleading 
that only this function can possibly update HW, instead I add in the comments 
for each function like expandISR and updateHW about which logic may triggers it.


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/DelayedFetch.scala, line 99
> > <https://reviews.apache.org/r/24676/diff/11/?file=724370#file724370line99>
> >
> >     I'm a bit confused by case C. It can also happen if the delayed fetch 
> > happens to straddle a segment roll event; the comment seems a bit 
> > misleading/incomplete without that.
> >     
> >     In fact, if it is lagging shouldn't it have been satisfied immediately 
> > without having to create a DelayedFetch in the first place?

It could be the case that it is lagging on one partition, but that alone cannot 
give enough data for the fetch.min.bytes since other partitions are all caught 
up. I reworded the comments a bit.


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/KafkaApis.scala, line 187
> > <https://reviews.apache.org/r/24676/diff/11/?file=724373#file724373line187>
> >
> >     Why is this additional logging necessary?
> >     
> >     KafkaApis currently has catch-all for unhandled exceptions.
> >     
> >     Error codes can be inspected via public access logs if required right?

The exception is already caught in the Replica manager, which does not re-throw 
but only set the error code. Hence the request log will not record this as an 
failed request.


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/KafkaApis.scala, line 423
> > <https://reviews.apache.org/r/24676/diff/11/?file=724373#file724373line423>
> >
> >     Are these changes intentional?

Yes. According to our logging wiki this should be debug level since they are 
not server side errors.


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/ReplicaManager.scala, line 46
> > <https://reviews.apache.org/r/24676/diff/11/?file=724376#file724376line46>
> >
> >     Should we rename ReplicaManager to ReplicatedLogManager?

I am going to do all the renaming in a follow-up JIRA.


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/KafkaApis.scala, line 261
> > <https://reviews.apache.org/r/24676/diff/11/?file=724373#file724373line261>
> >
> >     I'm not sure how scala treats this under the hood, but it _has_ to hold 
> > a reference to request until the callback is executed. i.e., we probably 
> > still want to empty the request data.

Good point!


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/ReplicaManager.scala, line 120
> > <https://reviews.apache.org/r/24676/diff/11/?file=724376#file724376line120>
> >
> >     (for regular consumer fetch)

Actually this is for both consumer / follower fetch


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/server/ReplicaManager.scala, line 265
> > <https://reviews.apache.org/r/24676/diff/11/?file=724376#file724376line265>
> >
> >     This is old code and we don't need to address it in this patch, but I 
> > was wondering if it makes sense to respond sooner if there is at least one 
> > error in the local append. What do you think? i.e., I don't remember a good 
> > reason for holding on to the request if there are i < numPartitions errors 
> > in local append.

I think today we are already responding immediately after a failure in local 
append, right?


> On Oct. 22, 2014, 1:46 a.m., Joel Koshy wrote:
> > core/src/main/scala/kafka/utils/DelayedItem.scala, line 23
> > <https://reviews.apache.org/r/24676/diff/11/?file=724378#file724378line23>
> >
> >     We don't really need this class anymore and it can be folded into 
> > DelayedRequest right?

I am going to do this in a follow-up JIRA.


- Guozhang


-----------------------------------------------------------
This is an automatically generated e-mail. To reply, visit:
https://reviews.apache.org/r/24676/#review57235
-----------------------------------------------------------


On Oct. 23, 2014, 1:53 a.m., Guozhang Wang wrote:
> 
> -----------------------------------------------------------
> This is an automatically generated e-mail. To reply, visit:
> https://reviews.apache.org/r/24676/
> -----------------------------------------------------------
> 
> (Updated Oct. 23, 2014, 1:53 a.m.)
> 
> 
> Review request for kafka.
> 
> 
> Bugs: KAFKA-1583
>     https://issues.apache.org/jira/browse/KAFKA-1583
> 
> 
> Repository: kafka
> 
> 
> Description
> -------
> 
> Incoporate Joel's comments after rebase
> 
> 
> Diffs
> -----
> 
>   core/src/main/scala/kafka/api/FetchRequest.scala 
> 59c09155dd25fad7bed07d3d00039e3dc66db95c 
>   core/src/main/scala/kafka/api/FetchResponse.scala 
> 8d085a1f18f803b3cebae4739ad8f58f95a6c600 
>   core/src/main/scala/kafka/api/OffsetCommitRequest.scala 
> 861a6cf11dc6b6431fcbbe9de00c74a122f204bd 
>   core/src/main/scala/kafka/api/ProducerRequest.scala 
> b2366e7eedcac17f657271d5293ff0bef6f3cbe6 
>   core/src/main/scala/kafka/api/ProducerResponse.scala 
> a286272c834b6f40164999ff8b7f8998875f2cfe 
>   core/src/main/scala/kafka/cluster/Partition.scala 
> e88ecf224a4dab8bbd26ba7b0c3ccfe844c6b7f4 
>   core/src/main/scala/kafka/common/ErrorMapping.scala 
> 880ab4a004f078e5d84446ea6e4454ecc06c95f2 
>   core/src/main/scala/kafka/log/Log.scala 
> 157d67369baabd2206a2356b2aa421e848adab17 
>   core/src/main/scala/kafka/network/BoundedByteBufferSend.scala 
> a624359fb2059340bb8dc1619c5b5f226e26eb9b 
>   core/src/main/scala/kafka/server/DelayedFetch.scala 
> e0f14e25af03e6d4344386dcabc1457ee784d345 
>   core/src/main/scala/kafka/server/DelayedProduce.scala 
> 9481508fc2d6140b36829840c337e557f3d090da 
>   core/src/main/scala/kafka/server/FetchRequestPurgatory.scala 
> ed1318891253556cdf4d908033b704495acd5724 
>   core/src/main/scala/kafka/server/KafkaApis.scala 
> 85498b4a1368d3506f19c4cfc64934e4d0ac4c90 
>   core/src/main/scala/kafka/server/OffsetManager.scala 
> 43eb2a35bb54d32c66cdb94772df657b3a104d1a 
>   core/src/main/scala/kafka/server/ProducerRequestPurgatory.scala 
> d4a7d4a79b44263a1f7e1a92874dd36aa06e5a3f 
>   core/src/main/scala/kafka/server/ReplicaManager.scala 
> 78b7514cc109547c562e635824684fad581af653 
>   core/src/main/scala/kafka/server/RequestPurgatory.scala 
> 9d76234bc2c810ec08621dc92bb4061b8e7cd993 
>   core/src/main/scala/kafka/utils/DelayedItem.scala 
> d7276494072f14f1cdf7d23f755ac32678c5675c 
>   core/src/test/scala/integration/kafka/api/ProducerFailureHandlingTest.scala 
> 209a409cb47eb24f83cee79f4e064dbc5f5e9d62 
>   core/src/test/scala/unit/kafka/producer/SyncProducerTest.scala 
> fb61d552f2320fedec547400fbbe402a0b2f5d87 
>   core/src/test/scala/unit/kafka/server/HighwatermarkPersistenceTest.scala 
> 03a424d45215e1e7780567d9559dae4d0ae6fc29 
>   core/src/test/scala/unit/kafka/server/ISRExpirationTest.scala 
> cd302aa51eb8377d88b752d48274e403926439f2 
>   core/src/test/scala/unit/kafka/server/ReplicaManagerTest.scala 
> a9c4ddc78df0b3695a77a12cf8cf25521a203122 
>   core/src/test/scala/unit/kafka/server/RequestPurgatoryTest.scala 
> a577f4a8bf420a5bc1e62fad6d507a240a42bcaa 
>   core/src/test/scala/unit/kafka/server/ServerShutdownTest.scala 
> 3804a114e97c849cae48308997037786614173fc 
>   core/src/test/scala/unit/kafka/server/SimpleFetchTest.scala 
> 09ed8f5a7a414ae139803bf82d336c2d80bf4ac5 
> 
> Diff: https://reviews.apache.org/r/24676/diff/
> 
> 
> Testing
> -------
> 
> Unit tests
> 
> 
> Thanks,
> 
> Guozhang Wang
> 
>

Reply via email to