[ https://issues.apache.org/jira/browse/KAFKA-1718?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14178796#comment-14178796 ]
Wojciech Kuranowski commented on KAFKA-1718: -------------------------------------------- My test is almost the same as in Description of this issue. But I have some interesting things to show. I print text "msg-before:" and MessageSet.entrySize(m) for every message in "analyzeAndValidateMessageSet" method. And then I print "msg-after:" and MessageSet.entrySize(m) for every message in "validMessages" collection in "append" method. As you can see, there are 4 messages "before" and just one "after". It looks like Broker decompress everything, squash into one message and then calculate size. I am not a Kafka expert, so this is my guessing. [2014-10-21 20:06:45,299] INFO msg-before: 452610 (kafka.log.Log) [2014-10-21 20:06:45,331] INFO msg-before: 450961 (kafka.log.Log) [2014-10-21 20:06:45,337] INFO msg-before: 449214 (kafka.log.Log) [2014-10-21 20:06:45,338] INFO msg-before: 384778 (kafka.log.Log) [2014-10-21 20:06:45,573] INFO msg-after: 3377212 (kafka.log.Log) [2014-10-21 20:06:45,577] ERROR [KafkaApi-1] Error processing ProducerRequest with correlation id 0 from client kafka-log-producer on partition [adstream2,0] (kafka.server.KafkaApis) kafka.common.MessageSizeTooLargeException: revalidate - Message size is 3377212 bytes which exceeds the maximum configured message size of 1000012. [2014-10-21 20:06:46,285] INFO msg-before: 464793 (kafka.log.Log) [2014-10-21 20:06:46,286] INFO msg-before: 465262 (kafka.log.Log) [2014-10-21 20:06:46,286] INFO msg-before: 465959 (kafka.log.Log) [2014-10-21 20:06:46,287] INFO msg-before: 380474 (kafka.log.Log) [2014-10-21 20:06:46,364] INFO msg-after: 3383301 (kafka.log.Log) [2014-10-21 20:06:46,365] ERROR [KafkaApi-1] Error processing ProducerRequest with correlation id 0 from client kafka-log-producer on partition [adstream2,0] (kafka.server.KafkaApis) kafka.common.MessageSizeTooLargeException: revalidate - Message size is 3383301 bytes which exceeds the maximum configured message size of 1000012. > "Message Size Too Large" error when only small messages produced with Snappy > ---------------------------------------------------------------------------- > > Key: KAFKA-1718 > URL: https://issues.apache.org/jira/browse/KAFKA-1718 > Project: Kafka > Issue Type: Bug > Components: core > Affects Versions: 0.8.1.1 > Reporter: Evan Huus > Priority: Critical > > I'm the primary author of the Go bindings, and while I originally received > this as a bug against my bindings, I'm coming to the conclusion that it's a > bug in the broker somehow. > Specifically, take a look at the last two kafka packets in the following > packet capture: https://dl.dropboxusercontent.com/u/171647/kafka.pcapng (you > will need a trunk build of Wireshark to fully decode the kafka part of the > packets). > The produce request contains two partitions on one topic. Each partition has > one message set (sizes 977205 bytes and 967362 bytes respectively). Each > message set is a sequential collection of snappy-compressed messages, each > message of size 46899. When uncompressed, each message contains a message set > of 999600 bytes, containing a sequence of uncompressed 1024-byte messages. > However, the broker responds to this with a MessageSizeTooLarge error, full > stacktrace from the broker logs being: > kafka.common.MessageSizeTooLargeException: Message size is 1070127 bytes > which exceeds the maximum configured message size of 1000012. > at kafka.log.Log$$anonfun$append$1.apply(Log.scala:267) > at kafka.log.Log$$anonfun$append$1.apply(Log.scala:265) > at scala.collection.Iterator$class.foreach(Iterator.scala:727) > at kafka.utils.IteratorTemplate.foreach(IteratorTemplate.scala:32) > at kafka.log.Log.append(Log.scala:265) > at kafka.cluster.Partition.appendMessagesToLeader(Partition.scala:354) > at > kafka.server.KafkaApis$$anonfun$appendToLocalLog$2.apply(KafkaApis.scala:376) > at > kafka.server.KafkaApis$$anonfun$appendToLocalLog$2.apply(KafkaApis.scala:366) > at > scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:244) > at > scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:244) > at > scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:98) > at > scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:98) > at > scala.collection.mutable.HashTable$class.foreachEntry(HashTable.scala:226) > at scala.collection.mutable.HashMap.foreachEntry(HashMap.scala:39) > at scala.collection.mutable.HashMap.foreach(HashMap.scala:98) > at scala.collection.TraversableLike$class.map(TraversableLike.scala:244) > at scala.collection.AbstractTraversable.map(Traversable.scala:105) > at kafka.server.KafkaApis.appendToLocalLog(KafkaApis.scala:366) > at kafka.server.KafkaApis.handleProducerRequest(KafkaApis.scala:292) > at kafka.server.KafkaApis.handle(KafkaApis.scala:185) > at kafka.server.KafkaRequestHandler.run(KafkaRequestHandler.scala:42) > at java.lang.Thread.run(Thread.java:695) > Since as far as I can tell none of the sizes in the actual produced packet > exceed the defined maximum, I can only assume that the broker is > miscalculating something somewhere and throwing the exception improperly. > --- > This issue can be reliably reproduced using an out-of-the-box binary download > of 0.8.1.1 and the following gist: > https://gist.github.com/eapache/ce0f15311c605a165ce7 (you will need to use > the `producer-ng` branch of the Sarama library). > --- > I am happy to provide any more information you might need, or to do relevant > experiments etc. -- This message was sent by Atlassian JIRA (v6.3.4#6332)