Could you try the latest code in the 0.8 branch? It fixed one consumer bug
that could cause the issue you saw.

Thanks,

Jun


On Thu, Jun 20, 2013 at 8:29 AM, Yu, Libo <libo...@citi.com> wrote:

> Sorry for my late reply. I used to watch the mailing list with Grokbase.
> It somehow stops to update the mailing list and I thought you guys
> were too busy to reply to us:)
> In my case, I have 6 topics ech with 6 partitions. The total data volume
> is about 20G.
>
> For my question 1), is it expected to have only a single
> partition after it is mirrored?
>
> To solve the issue in 1), a topic is created on source and destination at
> the same
> time with the same number of partitions. Then I noticed the consumers only
> fetch part of data then stop. Could you weigh in on that? Thanks.
>
>
> Regards,
>
> Libo
>
> From: Yu, Libo [ICG-IT]
> Sent: Monday, June 10, 2013 11:21 AM
> To: 'users@kafka.apache.org'
> Subject: rebalance issues from mirrormaker
>
> Hi,
>
> I come across several critical issues when benchmarking mirrormakers.
>
> 1 If a topic has N partitions on source cluster, after mirroring, on the
> destination
> cluster it has only one partition.
>
> 2 To solve the issue in 1), a topic is created on source and destination
> at the same
> time with the same number of partitions. After the topic is created on
> both clusters,
> some data is published to source cluster under the created topic. After
> publishing
> is done, multiple (3 in this case) mirrormaker processes are brought up on
> destination
> cluster to download data from the source. But the following exception
> occurs:
>
> [2013-06-10 10:38:14,955] INFO [libogroup_myhost-1370875086011-c2357858],
> Committing all offsets after clearing the fetcher queues
> (kafka.consumer.ZookeeperConsumerConnector)
> Exception in thread "main" kafka.common.ConsumerRebalanceFailedException:
> libogroup_ myhost--1370875086011-c2357858 can't rebalance after 4 retries
>         at
> kafka.consumer.ZookeeperConsumerConnector$ZKRebalancerListener.syncedRebalance(ZookeeperConsumerConnector.scala:397)
>         at
> kafka.consumer.ZookeeperConsumerConnector.kafka$consumer$ZookeeperConsumerConnector$$reinitializeConsumer(ZookeeperConsumerConnector.scala:680)
>         at
> kafka.consumer.ZookeeperConsumerConnector$WildcardStreamsHandler.<init>(ZookeeperConsumerConnector.scala:714)
>         at
> kafka.consumer.ZookeeperConsumerConnector.createMessageStreamsByFilter(ZookeeperConsumerConnector.scala:140)
>         at kafka.tools.MirrorMaker$$anonfun$4.apply(MirrorMaker.scala:118)
>         at kafka.tools.MirrorMaker$$anonfun$4.apply(MirrorMaker.scala:118)
>         at
> scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:206)
>         at
> scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:206)
>         at
> scala.collection.LinearSeqOptimized$class.foreach(LinearSeqOptimized.scala:61)
>         at scala.collection.immutable.List.foreach(List.scala:45)
>         at
> scala.collection.TraversableLike$class.map(TraversableLike.scala:206)
>         at scala.collection.immutable.List.map(List.scala:45)
>         at kafka.tools.MirrorMaker$.main(MirrorMaker.scala:118)
>         at kafka.tools.MirrorMaker.main(MirrorMaker.scala)
>
> 3) A topic is only created on source cluster. Some data is published to
> the source
>     cluster under the topic. Now launch multiple (3 in this case) consumer
> processes
>     at the same time, each of which has three stream. The exception also
> occurs:
>
> Exception in thread "main" [2013-06-10 10:48:28,673] INFO [libogroup_
> myhost -1370875697107-38deafbc], begin rebalancing consumer
> libogroup_myhost-1370875697107-38deafbc try #0
> (kafka.consumer.ZookeeperConsumerConnector)
> kafka.common.ConsumerRebalanceFailedException: libogroup_ myhost
> -1370875697107-38deafbc can't rebalance after 4 retries
>         at
> kafka.consumer.ZookeeperConsumerConnector$ZKRebalancerListener.syncedRebalance(ZookeeperConsumerConnector.scala:397)
>         at
> kafka.consumer.ZookeeperConsumerConnector.kafka$consumer$ZookeeperConsumerConnector$$reinitializeConsumer(ZookeeperConsumerConnector.scala:680)
>         at
> kafka.consumer.ZookeeperConsumerConnector$WildcardStreamsHandler.<init>(ZookeeperConsumerConnector.scala:714)
>         at
> kafka.consumer.ZookeeperConsumerConnector.createMessageStreamsByFilter(ZookeeperConsumerConnector.scala:140)
>         at kafka.tools.MirrorMaker$$anonfun$4.apply(MirrorMaker.scala:118)
>         at kafka.tools.MirrorMaker$$anonfun$4.apply(MirrorMaker.scala:118)
>         at
> scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:206)
>         at
> scala.collection.TraversableLike$$anonfun$map$1.apply(TraversableLike.scala:206)
>         at
> scala.collection.LinearSeqOptimized$class.foreach(LinearSeqOptimized.scala:61)
>         at scala.collection.immutable.List.foreach(List.scala:45)
>         at
> scala.collection.TraversableLike$class.map(TraversableLike.scala:206)
>         at scala.collection.immutable.List.map(List.scala:45)
>         at kafka.tools.MirrorMaker$.main(MirrorMaker.scala:118)
>         at kafka.tools.MirrorMaker.main(MirrorMaker.scala)
>
>
>
> Regards,
>
> Libo
>
>

Reply via email to