Re: C++ Client Library -- libkafka-asio

2015-03-23 Thread svante karlsson
@Ewen added license.txt (boost v1.0)

thanks
svante



2015-03-24 2:15 GMT+01:00 Ewen Cheslack-Postava :

> You don't get edit permission by default, you need to get one of the admins
> to give you permission.
>
> @Daniel, I've added libkafka-asio.
>
> @svante I started to add csi-kafka, but couldn't find a license?
>
>
> On Sun, Mar 22, 2015 at 8:29 AM, svante karlsson  wrote:
>
> > Cool, Looks nice. I was looking for something similar a year ago. We also
> > ended up rolling our own. https://github.com/bitbouncer/csi-kafka
> >
> >
> > Have you got any performance figures?
> >
> > /svante
> >
> > 2015-03-22 14:29 GMT+01:00 Daniel Joos :
> >
> > > Hello there,
> > >
> > > I'm currently working on a C++ client library, implementing the Kafka
> > > protocol using Boost ASIO.
> > > You can find the source code and some examples on github:
> > > https://github.com/danieljoos/libkafka-asio
> > >
> > > I tried to add it to the "Clients" section of the Kafka wiki, but
> either
> > > I'm to blind to see the "Edit" button, or I just don't have enough
> > > permissions to edit the page ;-)
> > > In case you like the library, it would be very nice, if someone with
> > > sufficient permissions for the wiki could add it there.
> > >
> > > Thanks.
> > > Best regards,
> > >
> > > Daniel
> > >
> > >
> >
>
>
>
> --
> Thanks,
> Ewen
>


Re: New Producer Questions/Feedback

2015-03-23 Thread Jun Rao
RecordAccumulator is actually not part of the public api since it's
internal. The public apis are only those in
http://kafka.apache.org/082/javadoc/index.html?org/apache/kafka/clients/producer/KafkaProducer.html

Thanks,

Jun

On Mon, Mar 23, 2015 at 9:23 PM, Grant Henke  wrote:

> Thanks for validating that. I was thinking of solving it in the same
> fashion. Though I was unsure if there was/would be a use case to have
> multiple CompressionTypes in the same RecordAccumulator since the API was
> originally created this way.
>
> I would be happy to file a jira and can take on making the change too.
> Since
> RecordAccumulator is part of the public api, should the KIP process be
> followed here as well?
>
> On Mon, Mar 23, 2015 at 10:58 PM, Jun Rao  wrote:
>
> > Hi, Grant,
> >
> > The append api seems indeed a bit weird. The compression type is a
> producer
> > level config. Instead of passing it in for each append, we probably
> should
> > just pass it in once during the creation RecordAccumulator. Could you
> file
> > a jira to track this?
> >
> > Thanks,
> >
> > Jun
> >
> > On Mon, Mar 23, 2015 at 7:16 PM, Grant Henke 
> wrote:
> >
> > > I am reading over the new producer code in an effort to understand the
> > > implementation more thoroughly and had some questions/feedback.
> > >
> > > Currently org.apache.kafka.clients.producer.internals.RecordAccumulator
> > > append method accepts the compressionType on a per record basis. It
> looks
> > > like the code would only work on a per batch basis because the
> > > CompressionType is only used when creating a new RecordBatch. My
> > > understanding is this should only support setting per batch at most. I
> > may
> > > have misread this though. Is there a time where setting per record
> would
> > > make sense?
> > >
> > > public RecordAppendResult append(TopicPartition tp, byte[] key,
> > byte[]
> > > value, CompressionType compression, Callback callback) throws
> > > InterruptedException;
> > >
> > > Why does org.apache.kafka.common.serialization.Serializer Interface
> > require
> > > a topic?  Is there a use case where serialization would change based on
> > > topic?
> > >
> > >public byte[] serialize(String topic, T data);
> > >
> > > Thank you,
> > > Grant
> > >
> > > --
> > > Grant Henke
> > > Solutions Consultant | Cloudera
> > > ghe...@cloudera.com | 920-980-8979
> > > twitter.com/ghenke  |
> > > linkedin.com/in/granthenke
> > >
> >
>
>
>
> --
> Grant Henke
> Solutions Consultant | Cloudera
> ghe...@cloudera.com | 920-980-8979
> twitter.com/ghenke  |
> linkedin.com/in/granthenke
>


Re: New Producer Questions/Feedback

2015-03-23 Thread Grant Henke
Thanks for validating that. I was thinking of solving it in the same
fashion. Though I was unsure if there was/would be a use case to have
multiple CompressionTypes in the same RecordAccumulator since the API was
originally created this way.

I would be happy to file a jira and can take on making the change too. Since
RecordAccumulator is part of the public api, should the KIP process be
followed here as well?

On Mon, Mar 23, 2015 at 10:58 PM, Jun Rao  wrote:

> Hi, Grant,
>
> The append api seems indeed a bit weird. The compression type is a producer
> level config. Instead of passing it in for each append, we probably should
> just pass it in once during the creation RecordAccumulator. Could you file
> a jira to track this?
>
> Thanks,
>
> Jun
>
> On Mon, Mar 23, 2015 at 7:16 PM, Grant Henke  wrote:
>
> > I am reading over the new producer code in an effort to understand the
> > implementation more thoroughly and had some questions/feedback.
> >
> > Currently org.apache.kafka.clients.producer.internals.RecordAccumulator
> > append method accepts the compressionType on a per record basis. It looks
> > like the code would only work on a per batch basis because the
> > CompressionType is only used when creating a new RecordBatch. My
> > understanding is this should only support setting per batch at most. I
> may
> > have misread this though. Is there a time where setting per record would
> > make sense?
> >
> > public RecordAppendResult append(TopicPartition tp, byte[] key,
> byte[]
> > value, CompressionType compression, Callback callback) throws
> > InterruptedException;
> >
> > Why does org.apache.kafka.common.serialization.Serializer Interface
> require
> > a topic?  Is there a use case where serialization would change based on
> > topic?
> >
> >public byte[] serialize(String topic, T data);
> >
> > Thank you,
> > Grant
> >
> > --
> > Grant Henke
> > Solutions Consultant | Cloudera
> > ghe...@cloudera.com | 920-980-8979
> > twitter.com/ghenke  |
> > linkedin.com/in/granthenke
> >
>



-- 
Grant Henke
Solutions Consultant | Cloudera
ghe...@cloudera.com | 920-980-8979
twitter.com/ghenke  | linkedin.com/in/granthenke


Re: New Producer Questions/Feedback

2015-03-23 Thread Jun Rao
Hi, Grant,

The append api seems indeed a bit weird. The compression type is a producer
level config. Instead of passing it in for each append, we probably should
just pass it in once during the creation RecordAccumulator. Could you file
a jira to track this?

Thanks,

Jun

On Mon, Mar 23, 2015 at 7:16 PM, Grant Henke  wrote:

> I am reading over the new producer code in an effort to understand the
> implementation more thoroughly and had some questions/feedback.
>
> Currently org.apache.kafka.clients.producer.internals.RecordAccumulator
> append method accepts the compressionType on a per record basis. It looks
> like the code would only work on a per batch basis because the
> CompressionType is only used when creating a new RecordBatch. My
> understanding is this should only support setting per batch at most. I may
> have misread this though. Is there a time where setting per record would
> make sense?
>
> public RecordAppendResult append(TopicPartition tp, byte[] key, byte[]
> value, CompressionType compression, Callback callback) throws
> InterruptedException;
>
> Why does org.apache.kafka.common.serialization.Serializer Interface require
> a topic?  Is there a use case where serialization would change based on
> topic?
>
>public byte[] serialize(String topic, T data);
>
> Thank you,
> Grant
>
> --
> Grant Henke
> Solutions Consultant | Cloudera
> ghe...@cloudera.com | 920-980-8979
> twitter.com/ghenke  |
> linkedin.com/in/granthenke
>


New Producer Questions/Feedback

2015-03-23 Thread Grant Henke
I am reading over the new producer code in an effort to understand the
implementation more thoroughly and had some questions/feedback.

Currently org.apache.kafka.clients.producer.internals.RecordAccumulator
append method accepts the compressionType on a per record basis. It looks
like the code would only work on a per batch basis because the
CompressionType is only used when creating a new RecordBatch. My
understanding is this should only support setting per batch at most. I may
have misread this though. Is there a time where setting per record would
make sense?

public RecordAppendResult append(TopicPartition tp, byte[] key, byte[]
value, CompressionType compression, Callback callback) throws
InterruptedException;

Why does org.apache.kafka.common.serialization.Serializer Interface require
a topic?  Is there a use case where serialization would change based on
topic?

   public byte[] serialize(String topic, T data);

Thank you,
Grant

-- 
Grant Henke
Solutions Consultant | Cloudera
ghe...@cloudera.com | 920-980-8979
twitter.com/ghenke  | linkedin.com/in/granthenke


Re: [Discussion] Using Client Requests and Responses in Server

2015-03-23 Thread Jun Rao
I think what you are saying is that in RequestChannel, we can start
generating header/body for new request types and leave requestObj null. For
existing requests, header/body will be null initially. Gradually, we can
migrate each type of requests by populating header/body, instead of
requestObj. This makes sense to me since it serves two purposes (1) not
polluting the code base with duplicated request/response objects for new
types of requests and (2) allowing the refactoring of existing requests to
be done in smaller pieces.

Could you try that approach and perhaps just migrate one existing request
type (e.g. HeartBeatRequest) as an example? We probably need to rewind the
buffer after reading the requestId when deserializing the header (since the
header includes the request id).

Thanks,

Jun

On Mon, Mar 23, 2015 at 4:52 PM, Gwen Shapira  wrote:

> I'm thinking of a different approach, that will not fix everything, but
> will allow adding new requests without code duplication (and therefore
> unblock KIP-4):
>
> RequestChannel.request currently takes a buffer and parses it into an "old"
> request object. Since the objects are byte-compatibly, we should be able to
> parse existing requests into both old and new objects. New requests will
> only be parsed into new objects.
>
> Basically:
> val requestId = buffer.getShort()
> if (requestId in keyToNameAndDeserializerMap) {
>requestObj = RequestKeys.deserializerForKey(requestId)(buffer)
>header: RequestHeader = RequestHeader.parse(buffer)
>body: Struct =
> ProtoUtils.currentRequestSchema(apiKey).read(buffer).asInstanceOf[Struct]
> } else {
>requestObj = null
> header: RequestHeader = RequestHeader.parse(buffer)
>body: Struct =
> ProtoUtils.currentRequestSchema(apiKey).read(buffer).asInstanceOf[Struct]
> }
>
> This way existing KafkaApis will keep working as normal. The new Apis can
> implement just the new header/body requests.
> We'll do the same on the send-side: BoundedByteBufferSend can have a
> constructor that takes header/body instead of just a response object.
>
> Does that make sense?
>
> Once we have this in, we can move to:
> * Adding the missing request/response to the client code
> * Replacing requests that can be replaced
>
> It will also make life easier by having us review and tests smaller chunks
> of work (the existing patch is *huge* , touches nearly every core component
> and I'm not done yet...)
>
> Gwen
>
>
>
>
> On Sun, Mar 22, 2015 at 10:24 PM, Jay Kreps  wrote:
>
> > Ack, yeah, forgot about that.
> >
> > It's not just a difference of wrappers. The server side actually sends
> the
> > bytes lazily using FileChannel.transferTo. We need to make it possible to
> > carry over that optimization. In some sense what we want to be able to do
> > is set a value to a Send instead of a ByteBuffer.
> >
> > Let me try to add that support to the protocol definition stuff, will
> > probably take me a few days to free up time.
> >
> > -Jay
> >
> > On Sun, Mar 22, 2015 at 7:44 PM, Gwen Shapira 
> > wrote:
> >
> > > In case anyone is still following this thread, I need a bit of help :)
> > >
> > > The old FetchResponse.PartitionData included a MessageSet object.
> > > The new FetchResponse.PartitionData includes a ByteBuffer.
> > >
> > > However, when we read from logs, we return a MessageSet, and as far as
> I
> > > can see, these can't be converted to ByteBuffers (at least not without
> > > copying their data).
> > >
> > > Did anyone consider how to reconcile the MessageSets with the new
> > > FetchResponse objects?
> > >
> > > Gwen
> > >
> > >
> > > On Sat, Mar 21, 2015 at 6:54 PM, Gwen Shapira 
> > > wrote:
> > >
> > > > Note: I'm also treating ZkUtils as if it was a public API (i.e.
> > > converting
> > > > objects that are returned into o.a.k.common equivalents but not
> > changing
> > > > ZkUtils itself).
> > > > I know its not public, but I suspect I'm not the only developer here
> > who
> > > > has tons of external code that uses it.
> > > >
> > > > Gwen
> > > >
> > > > On Wed, Mar 18, 2015 at 5:48 PM, Gwen Shapira  >
> > > > wrote:
> > > >
> > > >> We can't rip them out completely, unfortunately - the SimpleConsumer
> > > uses
> > > >> them.
> > > >>
> > > >> So we'll need conversion at some point. I'll try to make the
> > > >> conversion point "just before hitting a public API that we can't
> > > >> modify", and hopefully it won't look too arbitrary.
> > > >>
> > > >>
> > > >>
> > > >> On Wed, Mar 18, 2015 at 5:24 PM, Jay Kreps 
> > wrote:
> > > >> > I think either approach is okay in the short term. However our
> goal
> > > >> should
> > > >> > be to eventually get rid of that duplicate code, so if you are up
> > for
> > > >> just
> > > >> > ripping and cutting that may get us there sooner.
> > > >> >
> > > >> > -Jay
> > > >> >
> > > >> > On Wed, Mar 18, 2015 at 5:19 PM, Gwen Shapira <
> > gshap...@cloudera.com>
> > > >> wrote:
> > > >> >
> > > >> >> Thanks!
> > > >> >>
> > > >> >> Another clarification:
> > > >> >> 

Re: C++ Client Library -- libkafka-asio

2015-03-23 Thread Ewen Cheslack-Postava
You don't get edit permission by default, you need to get one of the admins
to give you permission.

@Daniel, I've added libkafka-asio.

@svante I started to add csi-kafka, but couldn't find a license?


On Sun, Mar 22, 2015 at 8:29 AM, svante karlsson  wrote:

> Cool, Looks nice. I was looking for something similar a year ago. We also
> ended up rolling our own. https://github.com/bitbouncer/csi-kafka
>
>
> Have you got any performance figures?
>
> /svante
>
> 2015-03-22 14:29 GMT+01:00 Daniel Joos :
>
> > Hello there,
> >
> > I'm currently working on a C++ client library, implementing the Kafka
> > protocol using Boost ASIO.
> > You can find the source code and some examples on github:
> > https://github.com/danieljoos/libkafka-asio
> >
> > I tried to add it to the "Clients" section of the Kafka wiki, but either
> > I'm to blind to see the "Edit" button, or I just don't have enough
> > permissions to edit the page ;-)
> > In case you like the library, it would be very nice, if someone with
> > sufficient permissions for the wiki could add it there.
> >
> > Thanks.
> > Best regards,
> >
> > Daniel
> >
> >
>



-- 
Thanks,
Ewen


Review Request 32422: Patch for KAFKA-1554

2015-03-23 Thread Mayuresh Gharat

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

Review request for kafka.


Bugs: KAFKA-1554
https://issues.apache.org/jira/browse/KAFKA-1554


Repository: kafka


Description
---

Check for corrupt index files and delete them. They can be rebuilt.


Diffs
-

  core/src/main/scala/kafka/log/Log.scala 
06b8ecc5d11a1acfbaf3c693c42bf3ce5b2cd86d 

Diff: https://reviews.apache.org/r/32422/diff/


Testing
---


Thanks,

Mayuresh Gharat



[jira] [Commented] (KAFKA-1554) Corrupt index found on clean startup

2015-03-23 Thread Mayuresh Gharat (JIRA)

[ 
https://issues.apache.org/jira/browse/KAFKA-1554?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14377009#comment-14377009
 ] 

Mayuresh Gharat commented on KAFKA-1554:


Created reviewboard https://reviews.apache.org/r/32422/diff/
 against branch origin/trunk

> Corrupt index found on clean startup
> 
>
> Key: KAFKA-1554
> URL: https://issues.apache.org/jira/browse/KAFKA-1554
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8.1
> Environment: ubuntu 12.04, oracle jdk 1.7
>Reporter: Alexis Midon
>Assignee: Mayuresh Gharat
>Priority: Critical
> Fix For: 0.9.0
>
> Attachments: KAFKA-1554.patch
>
>
> On a clean start up, corrupted index files are found.
> After investigations, it appears that some pre-allocated index files are not 
> "compacted" correctly and the end of the file is full of zeroes.
> As a result, on start up, the last relative offset is zero which yields an 
> offset equal to the base offset.
> The workaround is to delete all index files of size 10MB (the size of the 
> pre-allocated files), and restart. Index files will be re-created.
> {code}
> find $your_data_directory -size 10485760c -name *.index #-delete
> {code}
> This is issue might be related/similar to 
> https://issues.apache.org/jira/browse/KAFKA-1112
> {code}
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,696 
> INFO main kafka.server.KafkaServer.info - [Kafka Server 847605514], starting
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,698 
> INFO main kafka.server.KafkaServer.info - [Kafka Server 847605514], 
> Connecting to zookeeper on 
> zk-main0.XXX:2181,zk-main1.XXX:2181,zk-main2.:2181/production/kafka/main
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,708 
> INFO 
> ZkClient-EventThread-14-zk-main0.XXX.com:2181,zk-main1.XXX.com:2181,zk-main2.XXX.com:2181,zk-main3.XXX.com:2181,zk-main4.XXX.com:2181/production/kafka/main
>  org.I0Itec.zkclient.ZkEventThread.run - Starting ZkClient event thread.
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:zookeeper.version=3.3.3-1203054, built on 11/17/2011 05:47 GMT
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:host.name=i-6b948138.inst.aws.airbnb.com
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.version=1.7.0_55
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.vendor=Oracle Corporation
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.home=/usr/lib/jvm/jre-7-oracle-x64/jre
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.class.path=libs/snappy-java-1.0.5.jar:libs/scala-library-2.10.1.jar:libs/slf4j-api-1.7.2.jar:libs/jopt-simple-3.2.jar:libs/metrics-annotation-2.2.0.jar:libs/log4j-1.2.15.jar:libs/kafka_2.10-0.8.1.jar:libs/zkclient-0.3.jar:libs/zookeeper-3.3.4.jar:libs/metrics-core-2.2.0.jar
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.library.path=/usr/java/packages/lib/amd64:/usr/lib64:/lib64:/lib:/usr/lib
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.io.tmpdir=/tmp
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.compiler=
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.name=Linux
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.arch=amd64
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.version=3.2.0-61-virtual
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:user.name=kafka
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO mai

[jira] [Updated] (KAFKA-1554) Corrupt index found on clean startup

2015-03-23 Thread Mayuresh Gharat (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-1554?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Mayuresh Gharat updated KAFKA-1554:
---
Status: Patch Available  (was: Open)

> Corrupt index found on clean startup
> 
>
> Key: KAFKA-1554
> URL: https://issues.apache.org/jira/browse/KAFKA-1554
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8.1
> Environment: ubuntu 12.04, oracle jdk 1.7
>Reporter: Alexis Midon
>Assignee: Mayuresh Gharat
>Priority: Critical
> Fix For: 0.9.0
>
> Attachments: KAFKA-1554.patch
>
>
> On a clean start up, corrupted index files are found.
> After investigations, it appears that some pre-allocated index files are not 
> "compacted" correctly and the end of the file is full of zeroes.
> As a result, on start up, the last relative offset is zero which yields an 
> offset equal to the base offset.
> The workaround is to delete all index files of size 10MB (the size of the 
> pre-allocated files), and restart. Index files will be re-created.
> {code}
> find $your_data_directory -size 10485760c -name *.index #-delete
> {code}
> This is issue might be related/similar to 
> https://issues.apache.org/jira/browse/KAFKA-1112
> {code}
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,696 
> INFO main kafka.server.KafkaServer.info - [Kafka Server 847605514], starting
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,698 
> INFO main kafka.server.KafkaServer.info - [Kafka Server 847605514], 
> Connecting to zookeeper on 
> zk-main0.XXX:2181,zk-main1.XXX:2181,zk-main2.:2181/production/kafka/main
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,708 
> INFO 
> ZkClient-EventThread-14-zk-main0.XXX.com:2181,zk-main1.XXX.com:2181,zk-main2.XXX.com:2181,zk-main3.XXX.com:2181,zk-main4.XXX.com:2181/production/kafka/main
>  org.I0Itec.zkclient.ZkEventThread.run - Starting ZkClient event thread.
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:zookeeper.version=3.3.3-1203054, built on 11/17/2011 05:47 GMT
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:host.name=i-6b948138.inst.aws.airbnb.com
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.version=1.7.0_55
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.vendor=Oracle Corporation
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.home=/usr/lib/jvm/jre-7-oracle-x64/jre
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.class.path=libs/snappy-java-1.0.5.jar:libs/scala-library-2.10.1.jar:libs/slf4j-api-1.7.2.jar:libs/jopt-simple-3.2.jar:libs/metrics-annotation-2.2.0.jar:libs/log4j-1.2.15.jar:libs/kafka_2.10-0.8.1.jar:libs/zkclient-0.3.jar:libs/zookeeper-3.3.4.jar:libs/metrics-core-2.2.0.jar
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.library.path=/usr/java/packages/lib/amd64:/usr/lib64:/lib64:/lib:/usr/lib
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.io.tmpdir=/tmp
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.compiler=
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.name=Linux
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.arch=amd64
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.version=3.2.0-61-virtual
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:user.name=kafka
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:user.home=/srv/kafka
> 2014-07-11T00:53:17+

[jira] [Updated] (KAFKA-1554) Corrupt index found on clean startup

2015-03-23 Thread Mayuresh Gharat (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-1554?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Mayuresh Gharat updated KAFKA-1554:
---
Attachment: KAFKA-1554.patch

> Corrupt index found on clean startup
> 
>
> Key: KAFKA-1554
> URL: https://issues.apache.org/jira/browse/KAFKA-1554
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8.1
> Environment: ubuntu 12.04, oracle jdk 1.7
>Reporter: Alexis Midon
>Assignee: Mayuresh Gharat
>Priority: Critical
> Fix For: 0.9.0
>
> Attachments: KAFKA-1554.patch
>
>
> On a clean start up, corrupted index files are found.
> After investigations, it appears that some pre-allocated index files are not 
> "compacted" correctly and the end of the file is full of zeroes.
> As a result, on start up, the last relative offset is zero which yields an 
> offset equal to the base offset.
> The workaround is to delete all index files of size 10MB (the size of the 
> pre-allocated files), and restart. Index files will be re-created.
> {code}
> find $your_data_directory -size 10485760c -name *.index #-delete
> {code}
> This is issue might be related/similar to 
> https://issues.apache.org/jira/browse/KAFKA-1112
> {code}
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,696 
> INFO main kafka.server.KafkaServer.info - [Kafka Server 847605514], starting
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,698 
> INFO main kafka.server.KafkaServer.info - [Kafka Server 847605514], 
> Connecting to zookeeper on 
> zk-main0.XXX:2181,zk-main1.XXX:2181,zk-main2.:2181/production/kafka/main
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,708 
> INFO 
> ZkClient-EventThread-14-zk-main0.XXX.com:2181,zk-main1.XXX.com:2181,zk-main2.XXX.com:2181,zk-main3.XXX.com:2181,zk-main4.XXX.com:2181/production/kafka/main
>  org.I0Itec.zkclient.ZkEventThread.run - Starting ZkClient event thread.
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:zookeeper.version=3.3.3-1203054, built on 11/17/2011 05:47 GMT
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:host.name=i-6b948138.inst.aws.airbnb.com
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,714 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.version=1.7.0_55
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.vendor=Oracle Corporation
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.home=/usr/lib/jvm/jre-7-oracle-x64/jre
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.class.path=libs/snappy-java-1.0.5.jar:libs/scala-library-2.10.1.jar:libs/slf4j-api-1.7.2.jar:libs/jopt-simple-3.2.jar:libs/metrics-annotation-2.2.0.jar:libs/log4j-1.2.15.jar:libs/kafka_2.10-0.8.1.jar:libs/zkclient-0.3.jar:libs/zookeeper-3.3.4.jar:libs/metrics-core-2.2.0.jar
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,715 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.library.path=/usr/java/packages/lib/amd64:/usr/lib64:/lib64:/lib:/usr/lib
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.io.tmpdir=/tmp
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:java.compiler=
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.name=Linux
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,716 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.arch=amd64
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:os.version=3.2.0-61-virtual
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:user.name=kafka
> 2014-07-11T00:53:17+00:00 i-6b948138 local3.info 2014-07-11 - 00:53:17,717 
> INFO main org.apache.zookeeper.ZooKeeper.logEnv - Client 
> environment:user.home=/srv/kafka
> 2014-07-11T00:53:17+00:00 i-

[jira] [Commented] (KAFKA-1688) Add authorization interface and naive implementation

2015-03-23 Thread Prasad Mujumdar (JIRA)

[ 
https://issues.apache.org/jira/browse/KAFKA-1688?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14376977#comment-14376977
 ] 

Prasad Mujumdar commented on KAFKA-1688:


[~bosco] and [~parth.brahmbhatt] With my limited understanding of Kafka, the 
authorization interface design looks fine to me.

A couple of minor comments/suggestions:
- Hierarchical privileges
  The current proposed doesn't have hierarchical privileges. I am wondering if 
a global/cluster level namespace would make sense here. It's very useful for 
managing admin or superuser privileges. Also it provides a cleaner way to 
manage 'create topic' privilege. Without a global namespace, you'll need to 
grant create access on a topic that doesn't exist. In the database world for 
example, you would grant create access on a database to allow users to create 
tables under it.
- 'ALL' privilege which combines all the available privileges.


> Add authorization interface and naive implementation
> 
>
> Key: KAFKA-1688
> URL: https://issues.apache.org/jira/browse/KAFKA-1688
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Reporter: Jay Kreps
>Assignee: Parth Brahmbhatt
> Fix For: 0.8.3
>
>
> Add a PermissionManager interface as described here:
> https://cwiki.apache.org/confluence/display/KAFKA/Security
> (possibly there is a better name?)
> Implement calls to the PermissionsManager in KafkaApis for the main requests 
> (FetchRequest, ProduceRequest, etc). We will need to add a new error code and 
> exception to the protocol to indicate "permission denied".
> Add a server configuration to give the class you want to instantiate that 
> implements that interface. That class can define its own configuration 
> properties from the main config file.
> Provide a simple implementation of this interface which just takes a user and 
> ip whitelist and permits those in either of the whitelists to do anything, 
> and denies all others.
> Rather than writing an integration test for this class we can probably just 
> use this class for the TLS and SASL authentication testing.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


Re: [Discussion] Using Client Requests and Responses in Server

2015-03-23 Thread Gwen Shapira
I'm thinking of a different approach, that will not fix everything, but
will allow adding new requests without code duplication (and therefore
unblock KIP-4):

RequestChannel.request currently takes a buffer and parses it into an "old"
request object. Since the objects are byte-compatibly, we should be able to
parse existing requests into both old and new objects. New requests will
only be parsed into new objects.

Basically:
val requestId = buffer.getShort()
if (requestId in keyToNameAndDeserializerMap) {
   requestObj = RequestKeys.deserializerForKey(requestId)(buffer)
   header: RequestHeader = RequestHeader.parse(buffer)
   body: Struct =
ProtoUtils.currentRequestSchema(apiKey).read(buffer).asInstanceOf[Struct]
} else {
   requestObj = null
header: RequestHeader = RequestHeader.parse(buffer)
   body: Struct =
ProtoUtils.currentRequestSchema(apiKey).read(buffer).asInstanceOf[Struct]
}

This way existing KafkaApis will keep working as normal. The new Apis can
implement just the new header/body requests.
We'll do the same on the send-side: BoundedByteBufferSend can have a
constructor that takes header/body instead of just a response object.

Does that make sense?

Once we have this in, we can move to:
* Adding the missing request/response to the client code
* Replacing requests that can be replaced

It will also make life easier by having us review and tests smaller chunks
of work (the existing patch is *huge* , touches nearly every core component
and I'm not done yet...)

Gwen




On Sun, Mar 22, 2015 at 10:24 PM, Jay Kreps  wrote:

> Ack, yeah, forgot about that.
>
> It's not just a difference of wrappers. The server side actually sends the
> bytes lazily using FileChannel.transferTo. We need to make it possible to
> carry over that optimization. In some sense what we want to be able to do
> is set a value to a Send instead of a ByteBuffer.
>
> Let me try to add that support to the protocol definition stuff, will
> probably take me a few days to free up time.
>
> -Jay
>
> On Sun, Mar 22, 2015 at 7:44 PM, Gwen Shapira 
> wrote:
>
> > In case anyone is still following this thread, I need a bit of help :)
> >
> > The old FetchResponse.PartitionData included a MessageSet object.
> > The new FetchResponse.PartitionData includes a ByteBuffer.
> >
> > However, when we read from logs, we return a MessageSet, and as far as I
> > can see, these can't be converted to ByteBuffers (at least not without
> > copying their data).
> >
> > Did anyone consider how to reconcile the MessageSets with the new
> > FetchResponse objects?
> >
> > Gwen
> >
> >
> > On Sat, Mar 21, 2015 at 6:54 PM, Gwen Shapira 
> > wrote:
> >
> > > Note: I'm also treating ZkUtils as if it was a public API (i.e.
> > converting
> > > objects that are returned into o.a.k.common equivalents but not
> changing
> > > ZkUtils itself).
> > > I know its not public, but I suspect I'm not the only developer here
> who
> > > has tons of external code that uses it.
> > >
> > > Gwen
> > >
> > > On Wed, Mar 18, 2015 at 5:48 PM, Gwen Shapira 
> > > wrote:
> > >
> > >> We can't rip them out completely, unfortunately - the SimpleConsumer
> > uses
> > >> them.
> > >>
> > >> So we'll need conversion at some point. I'll try to make the
> > >> conversion point "just before hitting a public API that we can't
> > >> modify", and hopefully it won't look too arbitrary.
> > >>
> > >>
> > >>
> > >> On Wed, Mar 18, 2015 at 5:24 PM, Jay Kreps 
> wrote:
> > >> > I think either approach is okay in the short term. However our goal
> > >> should
> > >> > be to eventually get rid of that duplicate code, so if you are up
> for
> > >> just
> > >> > ripping and cutting that may get us there sooner.
> > >> >
> > >> > -Jay
> > >> >
> > >> > On Wed, Mar 18, 2015 at 5:19 PM, Gwen Shapira <
> gshap...@cloudera.com>
> > >> wrote:
> > >> >
> > >> >> Thanks!
> > >> >>
> > >> >> Another clarification:
> > >> >> The Common request/responses use slightly different infrastructure
> > >> >> objects: Node instead of Broker, TopicPartition instead of
> > >> >> TopicAndPartition and few more.
> > >> >>
> > >> >> I can write utilities to convert Node to Broker to minimize the
> scope
> > >> >> of the change.
> > >> >> Or I can start replacing Brokers with Nodes across the board.
> > >> >>
> > >> >> I'm currently taking the second approach - i.e, if MetadataRequest
> is
> > >> >> now returning Node, I'm changing the entire line of dependencies to
> > >> >> use Nodes instead of broker.
> > >> >>
> > >> >> Is this acceptable, or do we want to take a more minimal approach
> for
> > >> >> this patch and do a larger replacement as a follow up?
> > >> >>
> > >> >> Gwen
> > >> >>
> > >> >>
> > >> >>
> > >> >>
> > >> >> On Wed, Mar 18, 2015 at 3:32 PM, Jay Kreps 
> > >> wrote:
> > >> >> > Great.
> > >> >> >
> > >> >> > For (3) yeah I think we should just think through the end-to-end
> > >> pattern
> > >> >> > for these versioned requests since it seems like we will have a
> > >> number of
> > >> >> > them.

Re: [Discussion] Using Client Requests and Responses in Server

2015-03-23 Thread Jun Rao
The transferTo stuff is really specialized for sending a fetch response
from a broker. Since we can't get rid of the scala FetchResponse
immediately, we can probably keep the way that fetch responses are sent
(through FetchResponseSend) right now until the protocol definition is
extended.

Thanks,

Jun

On Sun, Mar 22, 2015 at 10:24 PM, Jay Kreps  wrote:

> Ack, yeah, forgot about that.
>
> It's not just a difference of wrappers. The server side actually sends the
> bytes lazily using FileChannel.transferTo. We need to make it possible to
> carry over that optimization. In some sense what we want to be able to do
> is set a value to a Send instead of a ByteBuffer.
>
> Let me try to add that support to the protocol definition stuff, will
> probably take me a few days to free up time.
>
> -Jay
>
> On Sun, Mar 22, 2015 at 7:44 PM, Gwen Shapira 
> wrote:
>
> > In case anyone is still following this thread, I need a bit of help :)
> >
> > The old FetchResponse.PartitionData included a MessageSet object.
> > The new FetchResponse.PartitionData includes a ByteBuffer.
> >
> > However, when we read from logs, we return a MessageSet, and as far as I
> > can see, these can't be converted to ByteBuffers (at least not without
> > copying their data).
> >
> > Did anyone consider how to reconcile the MessageSets with the new
> > FetchResponse objects?
> >
> > Gwen
> >
> >
> > On Sat, Mar 21, 2015 at 6:54 PM, Gwen Shapira 
> > wrote:
> >
> > > Note: I'm also treating ZkUtils as if it was a public API (i.e.
> > converting
> > > objects that are returned into o.a.k.common equivalents but not
> changing
> > > ZkUtils itself).
> > > I know its not public, but I suspect I'm not the only developer here
> who
> > > has tons of external code that uses it.
> > >
> > > Gwen
> > >
> > > On Wed, Mar 18, 2015 at 5:48 PM, Gwen Shapira 
> > > wrote:
> > >
> > >> We can't rip them out completely, unfortunately - the SimpleConsumer
> > uses
> > >> them.
> > >>
> > >> So we'll need conversion at some point. I'll try to make the
> > >> conversion point "just before hitting a public API that we can't
> > >> modify", and hopefully it won't look too arbitrary.
> > >>
> > >>
> > >>
> > >> On Wed, Mar 18, 2015 at 5:24 PM, Jay Kreps 
> wrote:
> > >> > I think either approach is okay in the short term. However our goal
> > >> should
> > >> > be to eventually get rid of that duplicate code, so if you are up
> for
> > >> just
> > >> > ripping and cutting that may get us there sooner.
> > >> >
> > >> > -Jay
> > >> >
> > >> > On Wed, Mar 18, 2015 at 5:19 PM, Gwen Shapira <
> gshap...@cloudera.com>
> > >> wrote:
> > >> >
> > >> >> Thanks!
> > >> >>
> > >> >> Another clarification:
> > >> >> The Common request/responses use slightly different infrastructure
> > >> >> objects: Node instead of Broker, TopicPartition instead of
> > >> >> TopicAndPartition and few more.
> > >> >>
> > >> >> I can write utilities to convert Node to Broker to minimize the
> scope
> > >> >> of the change.
> > >> >> Or I can start replacing Brokers with Nodes across the board.
> > >> >>
> > >> >> I'm currently taking the second approach - i.e, if MetadataRequest
> is
> > >> >> now returning Node, I'm changing the entire line of dependencies to
> > >> >> use Nodes instead of broker.
> > >> >>
> > >> >> Is this acceptable, or do we want to take a more minimal approach
> for
> > >> >> this patch and do a larger replacement as a follow up?
> > >> >>
> > >> >> Gwen
> > >> >>
> > >> >>
> > >> >>
> > >> >>
> > >> >> On Wed, Mar 18, 2015 at 3:32 PM, Jay Kreps 
> > >> wrote:
> > >> >> > Great.
> > >> >> >
> > >> >> > For (3) yeah I think we should just think through the end-to-end
> > >> pattern
> > >> >> > for these versioned requests since it seems like we will have a
> > >> number of
> > >> >> > them. The serialization code used as you described gets us to the
> > >> right
> > >> >> > Struct which the user would then wrap in something like
> > >> ProduceRequest.
> > >> >> > Presumably there would just be one ProduceRequest that would
> > >> internally
> > >> >> > fill in things like null or otherwise adapt the struct to a
> usable
> > >> >> object.
> > >> >> > On the response side we would have the version from the request
> to
> > >> use
> > >> >> for
> > >> >> > correct versioning. On question is whether this is enough or
> > whether
> > >> we
> > >> >> > need to have switches in KafkaApis to do things like
> > >> >> >if(produceRequest.version == 3)
> > >> >> >// do something
> > >> >> >else
> > >> >> >   // do something else
> > >> >> >
> > >> >> > Basically it would be good to be able to write a quick wiki that
> > was
> > >> like
> > >> >> > "how to add or modify a kafka api" that explained the right way
> to
> > >> do all
> > >> >> > this.
> > >> >> >
> > >> >> > I don't think any of this necessarily blocks this ticket since at
> > the
> > >> >> > moment we don't have tons of versions of requests out there.
> > >> >> >
> > >> >> > -Jay
> > >> >> >
> > >> >> >

[jira] [Updated] (KAFKA-2041) Add ability to specify a KeyClass for KafkaLog4jAppender

2015-03-23 Thread Benoy Antony (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-2041?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Benoy Antony updated KAFKA-2041:

Attachment: kafka-2041-001.patch

In the attached patch, a _Keyer_ trait is defined. This trait has a function 
which when given a value , returns the key.

The _KafkaLog4jAppender_ is modified to accept a log4j configuration for a 
KeyClass. which will be an implementation of _Keyer_.  The _Keyer_ 
implementation is instantiated and will be used while appending a message.


> Add ability to specify a KeyClass for KafkaLog4jAppender
> 
>
> Key: KAFKA-2041
> URL: https://issues.apache.org/jira/browse/KAFKA-2041
> Project: Kafka
>  Issue Type: Improvement
>  Components: producer 
>Reporter: Benoy Antony
>Assignee: Jun Rao
> Attachments: kafka-2041-001.patch
>
>
> KafkaLog4jAppender is the Log4j Appender to publish messages to Kafka. 
> Since there is no key or explicit partition number, the messages are sent to 
> random partitions. 
> In some cases, it is possible to derive a key from the message itself. 
> So it may be beneficial to enable KafkaLog4jAppender to accept KeyClass which 
> will provide a key for a given message.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Created] (KAFKA-2041) Add ability to specify a KeyClass for KafkaLog4jAppender

2015-03-23 Thread Benoy Antony (JIRA)
Benoy Antony created KAFKA-2041:
---

 Summary: Add ability to specify a KeyClass for KafkaLog4jAppender
 Key: KAFKA-2041
 URL: https://issues.apache.org/jira/browse/KAFKA-2041
 Project: Kafka
  Issue Type: Improvement
  Components: producer 
Reporter: Benoy Antony
Assignee: Jun Rao


KafkaLog4jAppender is the Log4j Appender to publish messages to Kafka. 
Since there is no key or explicit partition number, the messages are sent to 
random partitions. 

In some cases, it is possible to derive a key from the message itself. 
So it may be beneficial to enable KafkaLog4jAppender to accept KeyClass which 
will provide a key for a given message.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Updated] (KAFKA-1888) Add a "rolling upgrade" system test

2015-03-23 Thread Abhishek Nigam (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-1888?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Abhishek Nigam updated KAFKA-1888:
--
Attachment: KAFKA-1888_2015-03-23_11:54:25.patch

> Add a "rolling upgrade" system test
> ---
>
> Key: KAFKA-1888
> URL: https://issues.apache.org/jira/browse/KAFKA-1888
> Project: Kafka
>  Issue Type: Improvement
>  Components: system tests
>Reporter: Gwen Shapira
>Assignee: Abhishek Nigam
> Fix For: 0.9.0
>
> Attachments: KAFKA-1888_2015-03-23_11:54:25.patch
>
>
> To help test upgrades and compatibility between versions, it will be cool to 
> add a rolling-upgrade test to system tests:
> Given two versions (just a path to the jars?), check that you can do a
> rolling upgrade of the brokers from one version to another (using clients 
> from the old version) without losing data.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Updated] (KAFKA-1888) Add a "rolling upgrade" system test

2015-03-23 Thread Abhishek Nigam (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-1888?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Abhishek Nigam updated KAFKA-1888:
--
Status: Patch Available  (was: Open)

> Add a "rolling upgrade" system test
> ---
>
> Key: KAFKA-1888
> URL: https://issues.apache.org/jira/browse/KAFKA-1888
> Project: Kafka
>  Issue Type: Improvement
>  Components: system tests
>Reporter: Gwen Shapira
>Assignee: Abhishek Nigam
> Fix For: 0.9.0
>
> Attachments: KAFKA-1888_2015-03-23_11:54:25.patch
>
>
> To help test upgrades and compatibility between versions, it will be cool to 
> add a rolling-upgrade test to system tests:
> Given two versions (just a path to the jars?), check that you can do a
> rolling upgrade of the brokers from one version to another (using clients 
> from the old version) without losing data.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Commented] (KAFKA-1888) Add a "rolling upgrade" system test

2015-03-23 Thread Abhishek Nigam (JIRA)

[ 
https://issues.apache.org/jira/browse/KAFKA-1888?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14376399#comment-14376399
 ] 

Abhishek Nigam commented on KAFKA-1888:
---

Updated reviewboard https://reviews.apache.org/r/30809/diff/
 against branch origin/trunk

> Add a "rolling upgrade" system test
> ---
>
> Key: KAFKA-1888
> URL: https://issues.apache.org/jira/browse/KAFKA-1888
> Project: Kafka
>  Issue Type: Improvement
>  Components: system tests
>Reporter: Gwen Shapira
>Assignee: Abhishek Nigam
> Fix For: 0.9.0
>
> Attachments: KAFKA-1888_2015-03-23_11:54:25.patch
>
>
> To help test upgrades and compatibility between versions, it will be cool to 
> add a rolling-upgrade test to system tests:
> Given two versions (just a path to the jars?), check that you can do a
> rolling upgrade of the brokers from one version to another (using clients 
> from the old version) without losing data.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


Re: Review Request 30809: Patch for KAFKA-1888

2015-03-23 Thread Abhishek Nigam

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

(Updated March 23, 2015, 6:54 p.m.)


Review request for kafka.


Bugs: KAFKA-1888
https://issues.apache.org/jira/browse/KAFKA-1888


Repository: kafka


Description (updated)
---

Updated the RB with Gwen's comments, Beckett's comments and a subset of 
Guozhang's comments


Diffs (updated)
-

  bin/kafka-run-class.sh 881f578a8f5c796fe23415b978c1ad35869af76e 
  core/src/main/scala/kafka/tools/ContinuousValidationTest.java PRE-CREATION 
  core/src/main/scala/kafka/utils/ShutdownableThread.scala 
fc226c863095b7761290292cd8755cd7ad0f155c 
  system_test/broker_upgrade/bin/test-broker-upgrade.sh PRE-CREATION 

Diff: https://reviews.apache.org/r/30809/diff/


Testing
---

Scripted it to run 20 times without any failures.
Command-line: broker-upgrade/bin/test.sh  


Thanks,

Abhishek Nigam



[jira] [Created] (KAFKA-2040) Update documentation with the details of async producer

2015-03-23 Thread Navina Ramesh (JIRA)
Navina Ramesh created KAFKA-2040:


 Summary: Update documentation with the details of async producer
 Key: KAFKA-2040
 URL: https://issues.apache.org/jira/browse/KAFKA-2040
 Project: Kafka
  Issue Type: Bug
Reporter: Navina Ramesh


The changes in the java-based kafka producer design and api is not obvious in 
the current kafka documentation (on kafka.apache.org).
As much as I love reading the documentation in the code, I think it will be 
easier to read it from the website. 
I esp. found [~junrao] 's presentation comparing the old and new Kafka Producer 
architecture at the Kafka Meetup (http://www.ustream.tv/recorded/58109076) very 
useful. Please add it to the existing docs.



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Commented] (KAFKA-1507) Using GetOffsetShell against non-existent topic creates the topic unintentionally

2015-03-23 Thread Sriharsha Chintalapani (JIRA)

[ 
https://issues.apache.org/jira/browse/KAFKA-1507?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14376040#comment-14376040
 ] 

Sriharsha Chintalapani commented on KAFKA-1507:
---

[~jkreps] Since create/update/topic requests are part of KIP-4. Your proposal 
if the producer is throwing errors like UnknownTopicOrPartition users should 
catch this error and use AdminClient create a topic?. I still see a benefit of 
allowing users to pass in their required topic config( partitions, replication 
etcc) and if there is no topic exists send a createTopicRequest. If this is not 
desirable as per your suggestion we need to implement AdminClient?. In this 
case they can use AdminUtils and we should modify the AdminUtils send requests 
to broker instead of directly sending requests to zookeeper. This will also 
help KAFKA-1688 as all the create/update/delete requests will go through broker 
authorizer. Let me know if this what your thinking.

> Using GetOffsetShell against non-existent topic creates the topic 
> unintentionally
> -
>
> Key: KAFKA-1507
> URL: https://issues.apache.org/jira/browse/KAFKA-1507
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8.1.1
> Environment: centos
>Reporter: Luke Forehand
>Assignee: Sriharsha Chintalapani
>Priority: Minor
>  Labels: newbie
> Attachments: KAFKA-1507.patch, KAFKA-1507.patch, 
> KAFKA-1507_2014-07-22_10:27:45.patch, KAFKA-1507_2014-07-23_17:07:20.patch, 
> KAFKA-1507_2014-08-12_18:09:06.patch, KAFKA-1507_2014-08-22_11:06:38.patch, 
> KAFKA-1507_2014-08-22_11:08:51.patch
>
>
> A typo in using GetOffsetShell command can cause a
> topic to be created which cannot be deleted (because deletion is still in
> progress)
> ./kafka-run-class.sh kafka.tools.GetOffsetShell --broker-list
> kafka10:9092,kafka11:9092,kafka12:9092,kafka13:9092 --topic typo --time 1
> ./kafka-topics.sh --zookeeper stormqa1/kafka-prod --describe --topic typo
> Topic:typo  PartitionCount:8ReplicationFactor:1 Configs:
>  Topic: typo Partition: 0Leader: 10  Replicas: 10
>   Isr: 10
> ...



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Comment Edited] (KAFKA-2036) Consumer and broker have different networks

2015-03-23 Thread Arsenii Krasikov (JIRA)

[ 
https://issues.apache.org/jira/browse/KAFKA-2036?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14372199#comment-14372199
 ] 

Arsenii Krasikov edited comment on KAFKA-2036 at 3/23/15 1:58 PM:
--

simple solution with exhaustive search through all broker's addresses is in 
[patch|https://issues.apache.org/jira/secure/attachment/12706525/patch]


was (Author: nyaapa):
simple solution with exhaustive search through all broker's addresses is in 
[patch|https://issues.apache.org/jira/secure/attachment/12706058/patch]

> Consumer and broker have different networks
> ---
>
> Key: KAFKA-2036
> URL: https://issues.apache.org/jira/browse/KAFKA-2036
> Project: Kafka
>  Issue Type: Bug
>  Components: network
>Affects Versions: 0.8.2.1
> Environment: oracle java {7,8}, ipv6 only consumer, ipv4 + ipv6 broker
>Reporter: Arsenii Krasikov
>Assignee: Jun Rao
> Attachments: patch
>
>
> If broker is connected to several networks ( for example ipv6 and ipv4 ) and 
> not all of them are reachable to consumer then 
> {{kafka.network.BlockingChannel}} gives up to connect after the first 
> "Network is unreachable" error not triyng remaining networks



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Updated] (KAFKA-2036) Consumer and broker have different networks

2015-03-23 Thread Arsenii Krasikov (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-2036?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Arsenii Krasikov updated KAFKA-2036:

Attachment: (was: patch)

> Consumer and broker have different networks
> ---
>
> Key: KAFKA-2036
> URL: https://issues.apache.org/jira/browse/KAFKA-2036
> Project: Kafka
>  Issue Type: Bug
>  Components: network
>Affects Versions: 0.8.2.1
> Environment: oracle java {7,8}, ipv6 only consumer, ipv4 + ipv6 broker
>Reporter: Arsenii Krasikov
>Assignee: Jun Rao
> Attachments: patch
>
>
> If broker is connected to several networks ( for example ipv6 and ipv4 ) and 
> not all of them are reachable to consumer then 
> {{kafka.network.BlockingChannel}} gives up to connect after the first 
> "Network is unreachable" error not triyng remaining networks



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Updated] (KAFKA-2036) Consumer and broker have different networks

2015-03-23 Thread Arsenii Krasikov (JIRA)

 [ 
https://issues.apache.org/jira/browse/KAFKA-2036?page=com.atlassian.jira.plugin.system.issuetabpanels:all-tabpanel
 ]

Arsenii Krasikov updated KAFKA-2036:

Attachment: patch

simplified patch

> Consumer and broker have different networks
> ---
>
> Key: KAFKA-2036
> URL: https://issues.apache.org/jira/browse/KAFKA-2036
> Project: Kafka
>  Issue Type: Bug
>  Components: network
>Affects Versions: 0.8.2.1
> Environment: oracle java {7,8}, ipv6 only consumer, ipv4 + ipv6 broker
>Reporter: Arsenii Krasikov
>Assignee: Jun Rao
> Attachments: patch
>
>
> If broker is connected to several networks ( for example ipv6 and ipv4 ) and 
> not all of them are reachable to consumer then 
> {{kafka.network.BlockingChannel}} gives up to connect after the first 
> "Network is unreachable" error not triyng remaining networks



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


[jira] [Commented] (KAFKA-873) Consider replacing zkclient with curator (with zkclient-bridge)

2015-03-23 Thread Tommy Becker (JIRA)

[ 
https://issues.apache.org/jira/browse/KAFKA-873?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=14375884#comment-14375884
 ] 

Tommy Becker commented on KAFKA-873:


I think it would be nice.  The status of zkclient seems like a mystery and 
everyone else has moved on to Curator.  As for the bridge, it's still around 
but it doesn't work in the scenario where you're bridging a non-namespaced 
curator to a namespaced zkclient.

> Consider replacing zkclient with curator (with zkclient-bridge)
> ---
>
> Key: KAFKA-873
> URL: https://issues.apache.org/jira/browse/KAFKA-873
> Project: Kafka
>  Issue Type: Improvement
>Affects Versions: 0.8.0
>Reporter: Scott Clasen
>Assignee: Grant Henke
>
> If zkclient was replaced with curator and curator-x-zkclient-bridge it would 
> be initially a drop-in replacement
> https://github.com/Netflix/curator/wiki/ZKClient-Bridge
> With the addition of a few more props to ZkConfig, and a bit of code this 
> would open up the possibility of using ACLs in zookeeper (which arent 
> supported directly by zkclient), as well as integrating with netflix 
> exhibitor for those of us using that.
> Looks like KafkaZookeeperClient needs some love anyhow...



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)


zookeeper usage in kafka offset commit / fetch requests

2015-03-23 Thread svante karlsson
I'm using kafka 0.8.2.0

I'm working on a C++ client library and I'm adding consumer offset
management to the client. (https://github.com/bitbouncer/csi-kafka)

I know that the creation of zookeeper "paths" is not handled by kafkabroker
so I've manually created

/consumers/consumer_offset_sample/offsets

in zookeeper using a command line utility.

After this I'm able to get consumer metadata from kafka.

If I commit a consumer offset to an existing topic/partition
("perf-8-new/0") I see the following paths in zookeeper

/consumers/consumer_offset_sample
offsets
offsets/perf-8-new
offsets/perf-8-new/0
owners

I'm surprised as to why the committed values shows up in zookeeper since I
have no bindings to a zookeeper and the "offset.storage" property is a
consumer config. My initial understanding was that they were only written
to __consumer_offsets topic.

Finally, I change the previously committed value manually in zookeeper
/consumers/consumer_offset_sample/offsets/perf-8-new/0  -> 42

then that's what I get back in my get_consumer_offset() as well - so it
seems the zookeeper is involved in the offsets commits/fetches from brokers
point of view.

Is this the intended way or am I doing something wrong?

best regards
svante