You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@kafka.apache.org by Jay Kreps <ja...@gmail.com> on 2013/01/09 17:18:31 UTC

Re: Kafka 0.8 without replication - handle broker failure/availability

This is a good point. We have discussed this a little bit before. The key
constraint is that with replication factor 1 you can choose one of the
following: (1) high availability, (2) correct semantic partitioning. That
is to say, if a particular partition is unavailable you have no choice but
to give up and throw an error or else send the message elsewhere.

Obviously replication fixes this by just making the partitions highly
available.

It isn't really correct for us to choose one of these for the user. If they
are depending on partitioning, silently sending data elsewhere may be worse
then giving an error. So the user needs to somehow specify which behavior
they want.

Here is a JIRA where we can work out the details. I suspect this is a
blocker for 0.8:
https://issues.apache.org/jira/browse/KAFKA-691

As a work around in the meantime you can probably run with
replication--although it sounds like you don't really need it, it shouldn't
hurt.

-Jay


On Wed, Jan 9, 2013 at 2:38 AM, Maxime Brugidou
<ma...@gmail.com>wrote:

> Hello, I am currently testing the 0.8 branch (and it works quite well). We
> plan to not use the replication feature for now since we don't really need
> it, we can afford to lose data in case of unrecoverable failure from a
> broker.
>
> However, we really don't want to have producers/consumers fail if a broker
> is down. The ideal scenario (that was working on 0.7) is that producers
> would just produce to available partitions and consumers would consume from
> available partitions. If the broker comes back online, the consumer will
> catch up, if not we can decide to throw away the data.
>
> Is this feasible from 0.8? right now if i kill a broker it just makes
> everything fail...
>
> Multiple issues will come up:
> - Since now the partitions are set globally and never change, the
> availability of a topic vary depending on where the partitions are located
> - We would need tools to make sure topics are spread enough and rebalance
> them accordingly, (using the "DDL" i heard about, i'm not sure yet about
> how it works, i tried editing the json strings in zk, it somehow works, and
> there's the reassignment admin command too)
>
> That looks rather complicated, or maybe I'm missing something? The model
> that was used in 0.7 looked much easier to operate (it had drawbacks, and
> couldn't do intra-cluster replication, but at least the availability of the
> cluster was much higher).
>
> Thanks in advance for any help/clues,
>
> Maxime
>

Re: Kafka 0.8 without replication - handle broker failure/availability

Posted by Jay Kreps <ja...@gmail.com>.
But I think Maxime's point is valid. In the common case that you don't care
about every single message and are willing to tolerate a little loss 0.8
will seem like a pretty big step back.

I don't think the solution of just randomly partitioning works because you
will still produce 1/nth of your data to a dead broker which will lead to
timeouts or errors and will in any case be slow.

I think we are pretty hesitant to add functionality to 0.8, but it might be
worth thinking through the simplest thing that would make this work in 0.8.
I haven't looked at it, but I think we do automatically refresh metadata
when there is a failure, so maybe passing in the metadata to the producer
and making the RandomPartitioner make use of this properly would not be too
big of a change.

-Jay


On Wed, Jan 9, 2013 at 9:34 AM, Jun Rao <ju...@gmail.com> wrote:

> Maxime,
>
> First of all, in 0.8, you can choose to have a replication factor of 2. It
> just means that one can tolerate only one broker failure.
>
> Second, our producer logic supports retries on failure. If a message can't
> be sent, on retry, our default partitioner will select another random
> partition to send the message to. So, with a replication factor of 1,
> assuming that you have enough partitions (partitions are typically spread
> over all brokers) and enough retries, the message is likely to be
> delivered.
>
> To improve this, another possibility is to create a new type of partitioner
> that will route messages randomly to only those partitions with a leader.
> However, this likely requires a new interface of the partitioner. In
> addition to number of partitions, the partitioner needs to know whether
> each partition is available or not.
>
> Thanks,
>
> Jun
>
> On Wed, Jan 9, 2013 at 8:43 AM, Maxime Brugidou
> <ma...@gmail.com>wrote:
>
> > Thanks for your response. I think the work-around is not really
> acceptable
> > for me since it will consume 3x the resources (because replication of 3
> is
> > the minimum acceptable) and it will still make the cluster less available
> > anyway (unless i have only 3 brokers).
> >
> > The thing is that 0.7 was making the cluster 100% available (for my use
> > case, accepting data loss) as long a single broker was alive.
> >
> > A way to handle this would be to:
> > 1. Have a lot of partitions per topic (more than the # of brokers)
> > 2. Have something that rebalances the partitions and make sure a broker
> has
> > a at least a partition for each topic (to make every topic "available")
> > 3. Have a setting in the consumer/producer that say "I don't care about
> > partitioning, just produce/consume wherever you can"
> >
> > This is probably not simple to implement, I'll add these ideas in the
> JIRA
> > and will pursue the discussion there.
> >
> > Maxime
> >
> > On Wed, Jan 9, 2013 at 5:18 PM, Jay Kreps <ja...@gmail.com> wrote:
> >
> > > As a work around in the meantime you can probably run with
> > > replication--although it sounds like you don't really need it, it
> > shouldn't
> > > hurt.
> > >
> >
>

Re: Kafka 0.8 without replication - handle broker failure/availability

Posted by Jun Rao <ju...@gmail.com>.
Maxime,

First of all, in 0.8, you can choose to have a replication factor of 2. It
just means that one can tolerate only one broker failure.

Second, our producer logic supports retries on failure. If a message can't
be sent, on retry, our default partitioner will select another random
partition to send the message to. So, with a replication factor of 1,
assuming that you have enough partitions (partitions are typically spread
over all brokers) and enough retries, the message is likely to be
delivered.

To improve this, another possibility is to create a new type of partitioner
that will route messages randomly to only those partitions with a leader.
However, this likely requires a new interface of the partitioner. In
addition to number of partitions, the partitioner needs to know whether
each partition is available or not.

Thanks,

Jun

On Wed, Jan 9, 2013 at 8:43 AM, Maxime Brugidou
<ma...@gmail.com>wrote:

> Thanks for your response. I think the work-around is not really acceptable
> for me since it will consume 3x the resources (because replication of 3 is
> the minimum acceptable) and it will still make the cluster less available
> anyway (unless i have only 3 brokers).
>
> The thing is that 0.7 was making the cluster 100% available (for my use
> case, accepting data loss) as long a single broker was alive.
>
> A way to handle this would be to:
> 1. Have a lot of partitions per topic (more than the # of brokers)
> 2. Have something that rebalances the partitions and make sure a broker has
> a at least a partition for each topic (to make every topic "available")
> 3. Have a setting in the consumer/producer that say "I don't care about
> partitioning, just produce/consume wherever you can"
>
> This is probably not simple to implement, I'll add these ideas in the JIRA
> and will pursue the discussion there.
>
> Maxime
>
> On Wed, Jan 9, 2013 at 5:18 PM, Jay Kreps <ja...@gmail.com> wrote:
>
> > As a work around in the meantime you can probably run with
> > replication--although it sounds like you don't really need it, it
> shouldn't
> > hurt.
> >
>

Re: Kafka 0.8 without replication - handle broker failure/availability

Posted by Maxime Brugidou <ma...@gmail.com>.
Thanks for your response. I think the work-around is not really acceptable
for me since it will consume 3x the resources (because replication of 3 is
the minimum acceptable) and it will still make the cluster less available
anyway (unless i have only 3 brokers).

The thing is that 0.7 was making the cluster 100% available (for my use
case, accepting data loss) as long a single broker was alive.

A way to handle this would be to:
1. Have a lot of partitions per topic (more than the # of brokers)
2. Have something that rebalances the partitions and make sure a broker has
a at least a partition for each topic (to make every topic "available")
3. Have a setting in the consumer/producer that say "I don't care about
partitioning, just produce/consume wherever you can"

This is probably not simple to implement, I'll add these ideas in the JIRA
and will pursue the discussion there.

Maxime

On Wed, Jan 9, 2013 at 5:18 PM, Jay Kreps <ja...@gmail.com> wrote:

> As a work around in the meantime you can probably run with
> replication--although it sounds like you don't really need it, it shouldn't
> hurt.
>

Re: Kafka 0.8 without replication - handle broker failure/availability

Posted by Maxime Brugidou <ma...@gmail.com>.
Thanks for your response. I think the work-around is not really acceptable
for me since it will consume 3x the resources (because replication of 3 is
the minimum acceptable) and it will still make the cluster less available
anyway (unless i have only 3 brokers).

The thing is that 0.7 was making the cluster 100% available (for my use
case, accepting data loss) as long a single broker was alive.

A way to handle this would be to:
1. Have a lot of partitions per topic (more than the # of brokers)
2. Have something that rebalances the partitions and make sure a broker has
a at least a partition for each topic (to make every topic "available")
3. Have a setting in the consumer/producer that say "I don't care about
partitioning, just produce/consume wherever you can"

This is probably not simple to implement, I'll add these ideas in the JIRA
and will pursue the discussion there.

Maxime

On Wed, Jan 9, 2013 at 5:18 PM, Jay Kreps <ja...@gmail.com> wrote:

> As a work around in the meantime you can probably run with
> replication--although it sounds like you don't really need it, it shouldn't
> hurt.
>