Re: Make documentation part of new features acceptance criteria?

2013-07-11 Thread Cosmin Lehene
I like the release criteria idea. Perhaps add them to coding guide or the
developer section on wiki?

WRT feature completeness, I didn't think about having a doc for each one,
but rather updating the existing doc or the CHANGES.txt file (we don't
have one yet) with a note when adding new configurations, new interfaces
or new tools.
I think should be an awareness thing mostly.
Kafka's documentation is actually pretty decent, otherwise and the Coding
Guidelines are great.

I'm not sure if this would work for Kafka or not but you may want to look
at http://hbase.apache.org/book.html for an example of documentation which
gets versioned with the code.

Cosmin






On 7/10/13 7:15 PM, "Jay Kreps"  wrote:

>I like the idea of improving our documentation. Help is very much
>appreciated in this area (but of course the problem is that the people who
>experience the holes almost by definition can't fill them in). So even
>just
>pointing out areas that aren't covered is really helpful.
>
>We are in a sort of awkward stage this week because we have a 0.8 beta
>release but no detailed docs on its internals.
>
>WRT your specific proposals. I don't think we should do the documentation
>with each feature because I think that tends to lead to a bunch of little
>documents one for each change. I think we effectively get this out of
>JIRA+wiki today. This usually serves as a fairly complete design doc +
>commentary be others. It is pretty hard to get information out of this
>format for a new user, though.
>
>We do version control documentation but we can't physically version
>control
>it with the code because the code is in git and Apache only allows SVN as
>a
>mechanism for publishing to xxx.apache.org. :-(
>
>Instead what about this: we add a new release criteria for documentation
>completeness. It would be good to formalize the release criteria anyway.
>Informally they are something like
>1. Developers think it is feature complete
>2. Unit tests pass
>3. Integration/stress tests pass
>4. Some production usage
>It would be good to add to this list (5) documentation up-to-date and not
>do a release without this.
>
>It is debatable whether this should apply to beta releases, but probably
>it
>should. We can certainly apply it to the final 0.8 release if people are
>on
>board.
>
>-Jay
>
>
>
>On Wed, Jul 10, 2013 at 1:17 AM, Cosmin Lehene  wrote:
>
>> I'm not sure if there's already a guideline like this, but I wouldn't it
>> make sense to have it in order to keep documentation in sync with the
>>code?
>> Also, having this type of documentation as part of the codebase to allow
>> proper versioning might be a good idea as well.
>>
>> Cosmin
>>



[jira] [Created] (KAFKA-972) MetadataRequest returns stale list of brokers

2013-07-11 Thread Vinicius Carvalho (JIRA)
Vinicius Carvalho created KAFKA-972:
---

 Summary: MetadataRequest returns stale list of brokers
 Key: KAFKA-972
 URL: https://issues.apache.org/jira/browse/KAFKA-972
 Project: Kafka
  Issue Type: Bug
  Components: core
Affects Versions: 0.8
Reporter: Vinicius Carvalho
 Fix For: 0.8


When we issue an metadatarequest towards the cluster, the list of brokers is 
stale. I mean, even when a broker is down, it's returned back to the client. 
The following are examples of two invocations one with both brokers online and 
the second with a broker down:

{
"brokers": [
{
"nodeId": 0,
"host": "10.139.245.106",
"port": 9092,
"byteLength": 24
},
{
"nodeId": 1,
"host": "localhost",
"port": 9093,
"byteLength": 19
}
],
"topicMetadata": [
{
"topicErrorCode": 0,
"topicName": "foozbar",
"partitions": [
{
"replicas": [
0
],
"isr": [
0
],
"partitionErrorCode": 0,
"partitionId": 0,
"leader": 0,
"byteLength": 26
},
{
"replicas": [
1
],
"isr": [
1
],
"partitionErrorCode": 0,
"partitionId": 1,
"leader": 1,
"byteLength": 26
},
{
"replicas": [
0
],
"isr": [
0
],
"partitionErrorCode": 0,
"partitionId": 2,
"leader": 0,
"byteLength": 26
},
{
"replicas": [
1
],
"isr": [
1
],
"partitionErrorCode": 0,
"partitionId": 3,
"leader": 1,
"byteLength": 26
},
{
"replicas": [
0
],
"isr": [
0
],
"partitionErrorCode": 0,
"partitionId": 4,
"leader": 0,
"byteLength": 26
}
],
"byteLength": 145
}
],
"responseSize": 200,
"correlationId": -1000
}


{
"brokers": [
{
"nodeId": 0,
"host": "10.139.245.106",
"port": 9092,
"byteLength": 24
},
{
"nodeId": 1,
"host": "localhost",
"port": 9093,
"byteLength": 19
}
],
"topicMetadata": [
{
"topicErrorCode": 0,
"topicName": "foozbar",
"partitions": [
{
"replicas": [
0
],
"isr": [],
"partitionErrorCode": 5,
"partitionId": 0,
"leader": -1,
"byteLength": 22
},
{
"replicas": [
1
],
"isr": [
1
],
"partitionErrorCode": 0,
"partitionId": 1,
"leader": 1,
"byteLength": 26
},
{
"replicas": [
0
],
"isr": [],
"partitionErrorCode": 5,
"partitionId": 2,
"leader": -1,
"byteLength": 22
},
{
"replicas": [
1
],
"isr": [
1
],
"partitionErrorCode": 0,
"partitionId": 3,
"leader": 1,
"byteLength": 26
},
{
"replicas": [
0
],
"isr": [],
"partitionErrorCode": 5,
"partitionId": 4,
"leader": -1,
   

[jira] [Updated] (KAFKA-615) Avoid fsync on log segment roll

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-615:


Attachment: KAFKA-615-v3.patch

Patch version v3:
- Found a call to flush the index in Log.roll(). Removed this.

> Avoid fsync on log segment roll
> ---
>
> Key: KAFKA-615
> URL: https://issues.apache.org/jira/browse/KAFKA-615
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: KAFKA-615-v1.patch, KAFKA-615-v2.patch, 
> KAFKA-615-v3.patch
>
>
> It still isn't feasible to run without an application level fsync policy. 
> This is a problem as fsync locks the file and tuning such a policy so that 
> the flushes aren't so frequent that seeks reduce throughput, yet not so 
> infrequent that the fsync is writing so much data that there is a noticable 
> jump in latency is very challenging.
> The remaining problem is the way that log recovery works. Our current policy 
> is that if a clean shutdown occurs we do no recovery. If an unclean shutdown 
> occurs we recovery the last segment of all logs. To make this correct we need 
> to ensure that each segment is fsync'd before we create a new segment. Hence 
> the fsync during roll.
> Obviously if the fsync during roll is the only time fsync occurs then it will 
> potentially write out the entire segment which for a 1GB segment at 50mb/sec 
> might take many seconds. The goal of this JIRA is to eliminate this and make 
> it possible to run with no application-level fsyncs at all, depending 
> entirely on replication and background writeback for durability.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-969) Need to prevent failure of rebalance when there are no brokers available when consumer comes up

2013-07-11 Thread Guozhang Wang (JIRA)

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

Guozhang Wang commented on KAFKA-969:
-

I think this is related to how we would enforce the order of starting 
servers/clients. In the quickstart Wiki page

http://kafka.apache.org/08/quickstart.html

We suggest to 1) start servers, 2) start producers and start sending messages, 
and then 3) start consumers. This is the normal case for production, whereas 
for dev testing this order would be different, such as 1) start servers with no 
topics created, 2) start producers and consumers, 3) producer start sending 
messages; or start 1) and 2) at roughly the same time; or even start 2) before 
start 1). When such order happens, these issues will arise.

> Need to prevent failure of rebalance when there are no brokers available when 
> consumer comes up
> ---
>
> Key: KAFKA-969
> URL: https://issues.apache.org/jira/browse/KAFKA-969
> Project: Kafka
>  Issue Type: Bug
>Reporter: Sriram Subramanian
>Assignee: Sriram Subramanian
> Fix For: 0.8
>
> Attachments: emptybrokeronrebalance-1.patch
>
>
> There are some rare instances when a consumer would be up before bringing up 
> the Kafka brokers. This would usually happen in a test scenario. In such 
> conditions, during rebalance instead of failing the rebalance we just log the 
> error and subscribe to broker changes. When the broker comes back up, we 
> trigger the rebalance.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Closed] (KAFKA-969) Need to prevent failure of rebalance when there are no brokers available when consumer comes up

2013-07-11 Thread Sriram Subramanian (JIRA)

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

Sriram Subramanian closed KAFKA-969.



> Need to prevent failure of rebalance when there are no brokers available when 
> consumer comes up
> ---
>
> Key: KAFKA-969
> URL: https://issues.apache.org/jira/browse/KAFKA-969
> Project: Kafka
>  Issue Type: Bug
>Reporter: Sriram Subramanian
>Assignee: Sriram Subramanian
> Fix For: 0.8
>
> Attachments: emptybrokeronrebalance-1.patch
>
>
> There are some rare instances when a consumer would be up before bringing up 
> the Kafka brokers. This would usually happen in a test scenario. In such 
> conditions, during rebalance instead of failing the rebalance we just log the 
> error and subscribe to broker changes. When the broker comes back up, we 
> trigger the rebalance.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-969) Need to prevent failure of rebalance when there are no brokers available when consumer comes up

2013-07-11 Thread Sam Meder (JIRA)

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

Sam Meder commented on KAFKA-969:
-

I feel that Kafka servers and clients should really not have to care about 
their respective start-up order and should be able to recover from any order. 
Otherwise certain recovery scenarios become cumbersome and it introduces a lot 
of extra complexity in the deployment system.

> Need to prevent failure of rebalance when there are no brokers available when 
> consumer comes up
> ---
>
> Key: KAFKA-969
> URL: https://issues.apache.org/jira/browse/KAFKA-969
> Project: Kafka
>  Issue Type: Bug
>Reporter: Sriram Subramanian
>Assignee: Sriram Subramanian
> Fix For: 0.8
>
> Attachments: emptybrokeronrebalance-1.patch
>
>
> There are some rare instances when a consumer would be up before bringing up 
> the Kafka brokers. This would usually happen in a test scenario. In such 
> conditions, during rebalance instead of failing the rebalance we just log the 
> error and subscribe to broker changes. When the broker comes back up, we 
> trigger the rebalance.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Re: having problem with 0.8 gzip compression

2013-07-11 Thread Scott Wang
Joel,

Would you mind point me to how I would be able to enable the trace logs in
the producer and broker?

Thanks,
Scott


On Wed, Jul 10, 2013 at 5:33 PM, Joel Koshy  wrote:

> Weird - I tried your exact code and it worked for me (although I was
> using 0.8 head and not the beta). Can you re-run with trace logs
> enabled in your producer and paste that output? Broker logs also if
> you can?
>
> Thanks,
>
> Joel
>
> On Wed, Jul 10, 2013 at 10:23 AM, Scott Wang
>  wrote:
> > Jun,
> >
> > I did a test this morning and got a very interesting result with you
> > command.  I started by wipe all the log files and clean up all zookeeper
> > data files.
> >
> > Once I restarted both server, producer and consumer then execute your
> > command, what I got is a empty log as following:
> >
> > Dumping /Users/scott/Temp/kafka/test-topic-0/.log
> > Starting offset: 0
> >
> > One observation, the .index file was getting huge but
> > there was nothing in .log file.
> >
> > Thanks,
> > Scott
> >
> >
> >
> >
> > On Tue, Jul 9, 2013 at 8:40 PM, Jun Rao  wrote:
> >
> >> Could you run the following command on one of the log files of your
> topic
> >> and attach the output?
> >>
> >> bin/kafka-run-class.sh kafka.tools.DumpLogSegments --files
> >> /tmp/kafka-logs/testtopic-0/.log
> >>
> >> Thanks,
> >>
> >> Jun
> >>
> >>
> >> On Tue, Jul 9, 2013 at 3:23 PM, Scott Wang <
> >> scott.w...@rumbleentertainment.com> wrote:
> >>
> >> > Another piece of information, the snappy compression also does not
> work.
> >> >
> >> > Thanks,
> >> > Scott
> >> >
> >> >
> >> > On Tue, Jul 9, 2013 at 11:07 AM, Scott Wang <
> >> > scott.w...@rumbleentertainment.com> wrote:
> >> >
> >> > > I just try it and it still not showing up, thanks for looking into
> >> this.
> >> > >
> >> > > Thanks,
> >> > > Scott
> >> > >
> >> > >
> >> > > On Tue, Jul 9, 2013 at 8:06 AM, Jun Rao  wrote:
> >> > >
> >> > >> Could you try starting the consumer first (and enable gzip in the
> >> > >> producer)?
> >> > >>
> >> > >> Thanks,
> >> > >>
> >> > >> Jun
> >> > >>
> >> > >>
> >> > >> On Mon, Jul 8, 2013 at 9:37 PM, Scott Wang <
> >> > >> scott.w...@rumbleentertainment.com> wrote:
> >> > >>
> >> > >> > No, I did not start the consumer before the producer.  I actually
> >> > >> started
> >> > >> > the producer first and nothing showed up in the consumer unless I
> >> > >> commented
> >> > >> > out this line -- props.put("compression.codec", "gzip").If I
> >> > >> commented
> >> > >> > out the compression codec, everything just works.
> >> > >> >
> >> > >> >
> >> > >> > On Mon, Jul 8, 2013 at 9:07 PM, Jun Rao 
> wrote:
> >> > >> >
> >> > >> > > Did you start the consumer before the producer? Be default, the
> >> > >> consumer
> >> > >> > > gets only the new data?
> >> > >> > >
> >> > >> > > Thanks,
> >> > >> > >
> >> > >> > > Jun
> >> > >> > >
> >> > >> > >
> >> > >> > > On Mon, Jul 8, 2013 at 2:53 PM, Scott Wang <
> >> > >> > > scott.w...@rumbleentertainment.com> wrote:
> >> > >> > >
> >> > >> > > > I am testing with Kafka 0.8 beta and having problem of
> receiving
> >> > >> > message
> >> > >> > > in
> >> > >> > > > consumer.  There is no error so does anyone have any
> insights.
> >> > >>  When I
> >> > >> > > > commented out the "compression.code" everything works fine.
> >> > >> > > >
> >> > >> > > > My producer:
> >> > >> > > > public class TestKafka08Prod {
> >> > >> > > >
> >> > >> > > > public static void main(String [] args) {
> >> > >> > > >
> >> > >> > > > Producer producer = null;
> >> > >> > > > try {
> >> > >> > > > Properties props = new Properties();
> >> > >> > > > props.put("metadata.broker.list",
> "localhost:9092");
> >> > >> > > > props.put("serializer.class",
> >> > >> > > > "kafka.serializer.StringEncoder");
> >> > >> > > > props.put("producer.type", "sync");
> >> > >> > > > props.put("request.required.acks","1");
> >> > >> > > > props.put("compression.codec", "gzip");
> >> > >> > > > ProducerConfig config = new
> ProducerConfig(props);
> >> > >> > > > producer = new Producer(config);
> >> > >> > > > int j=0;
> >> > >> > > > for(int i=0; i<10; i++) {
> >> > >> > > > KeyedMessage data = new
> >> > >> > > > KeyedMessage("test-topic", "test-message:
> "+i+"
> >> > >> > > > "+System.currentTimeMillis());
> >> > >> > > > producer.send(data);
> >> > >> > > >
> >> > >> > > > }
> >> > >> > > >
> >> > >> > > > } catch (Exception e) {
> >> > >> > > > System.out.println("Error happened: ");
> >> > >> > > > e.printStackTrace();
> >> > >> > > > } finally {
> >> > >> > > > if(null != null) {
> >> > >> > > > producer.close();
> >> > >> > > > }
> >> > >> > > >
> >> > >> > > > System.out.println("Ened of Sending"

[jira] Subscription: outstanding kafka patches

2013-07-11 Thread jira
Issue Subscription
Filter: outstanding kafka patches (75 issues)
The list of outstanding kafka patches
Subscriber: kafka-mailing-list

Key Summary
KAFKA-967   Use key range in ProducerPerformance
https://issues.apache.org/jira/browse/KAFKA-967
KAFKA-956   High-level consumer fails to check topic metadata response for 
errors
https://issues.apache.org/jira/browse/KAFKA-956
KAFKA-946   Kafka Hadoop Consumer fails when verifying message checksum
https://issues.apache.org/jira/browse/KAFKA-946
KAFKA-943   Move all configuration key string to constants
https://issues.apache.org/jira/browse/KAFKA-943
KAFKA-932   System Test - set retry.backoff.ms=300 to all test cases
https://issues.apache.org/jira/browse/KAFKA-932
KAFKA-925   Add optional partition key override in producer
https://issues.apache.org/jira/browse/KAFKA-925
KAFKA-923   Improve controller failover latency
https://issues.apache.org/jira/browse/KAFKA-923
KAFKA-922   System Test - set retry.backoff.ms=300 to testcase_0119
https://issues.apache.org/jira/browse/KAFKA-922
KAFKA-917   Expose zk.session.timeout.ms in console consumer
https://issues.apache.org/jira/browse/KAFKA-917
KAFKA-915   System Test - Mirror Maker testcase_5001 failed
https://issues.apache.org/jira/browse/KAFKA-915
KAFKA-898   Add a KafkaMetricsReporter that wraps Librato's reporter
https://issues.apache.org/jira/browse/KAFKA-898
KAFKA-885   sbt package builds two kafka jars
https://issues.apache.org/jira/browse/KAFKA-885
KAFKA-883   System Test - update migration tool testsuite after 0.7 
ProducerPerformance sends seq MessageID
https://issues.apache.org/jira/browse/KAFKA-883
KAFKA-882   Enhance 0.7 ProducerPerformance to send sequential MessageID as in 
0.8
https://issues.apache.org/jira/browse/KAFKA-882
KAFKA-881   Kafka broker not respecting log.roll.hours
https://issues.apache.org/jira/browse/KAFKA-881
KAFKA-879   In system test, read the new leader from zookeeper instead of 
broker log on completion of become-leader state transition
https://issues.apache.org/jira/browse/KAFKA-879
KAFKA-877   Still getting kafka.common.NotLeaderForPartitionException
https://issues.apache.org/jira/browse/KAFKA-877
KAFKA-873   Consider replacing zkclient with curator (with zkclient-bridge)
https://issues.apache.org/jira/browse/KAFKA-873
KAFKA-868   System Test - add test case for rolling controlled shutdown
https://issues.apache.org/jira/browse/KAFKA-868
KAFKA-863   System Test - update 0.7 version of kafka-run-class.sh for 
Migration Tool test cases
https://issues.apache.org/jira/browse/KAFKA-863
KAFKA-859   support basic auth protection of mx4j console
https://issues.apache.org/jira/browse/KAFKA-859
KAFKA-855   Ant+Ivy build for Kafka
https://issues.apache.org/jira/browse/KAFKA-855
KAFKA-854   Upgrade dependencies for 0.8
https://issues.apache.org/jira/browse/KAFKA-854
KAFKA-852   Remove clientId from OffsetFetchResponse and OffsetCommitResponse
https://issues.apache.org/jira/browse/KAFKA-852
KAFKA-836   Update quickstart for Kafka 0.8
https://issues.apache.org/jira/browse/KAFKA-836
KAFKA-815   Improve SimpleConsumerShell to take in a max messages config option
https://issues.apache.org/jira/browse/KAFKA-815
KAFKA-745   Remove getShutdownReceive() and other kafka specific code from the 
RequestChannel
https://issues.apache.org/jira/browse/KAFKA-745
KAFKA-735   Add looping and JSON output for ConsumerOffsetChecker
https://issues.apache.org/jira/browse/KAFKA-735
KAFKA-718   kafka-run-class.sh should use reasonable gc settings
https://issues.apache.org/jira/browse/KAFKA-718
KAFKA-717   scala 2.10 build support
https://issues.apache.org/jira/browse/KAFKA-717
KAFKA-705   Controlled shutdown doesn't seem to work on more than one broker in 
a cluster
https://issues.apache.org/jira/browse/KAFKA-705
KAFKA-686   0.8 Kafka broker should give a better error message when running 
against 0.7 zookeeper
https://issues.apache.org/jira/browse/KAFKA-686
KAFKA-682   java.lang.OutOfMemoryError: Java heap space
https://issues.apache.org/jira/browse/KAFKA-682
KAFKA-677   Retention process gives exception if an empty segment is chosen for 
collection
https://issues.apache.org/jira/browse/KAFKA-677
KAFKA-674   Clean Shutdown Testing - Log segments checksums mismatch
https://issues.apache.org/jira/browse/KAFKA-674
KAFKA-652   Create testcases for clean shut-down
https://issues.apache.org/jira/browse/KAFKA-652
KAFKA-649   Cleanup log4j logging
https://issues.apache.org/jira/browse/KAFKA-649
KAFKA-645   Create a shell script to run System Test with DEBUG details and 
"tee" 

Added an "operations" page for 0.8 to site

2013-07-11 Thread Jay Kreps
Probably it is very incomplete. I copied over much of the content from 0.7
that is still applicable, but I added the following:
1. A section on data centers and mirroring
2. Removed the cut-and-pasted scala code in the monitoring section and just
gave a high-level description of what we monitor
3. Added a lot of details about what we have learned about Linux, ext3,
write latency, etc.

http://kafka.apache.org/08/ops.html

Let me know any thoughts on how to improve this.

-Jay


[jira] [Updated] (KAFKA-718) kafka-run-class.sh should use reasonable gc settings

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-718:


Attachment: KAFKA-718-v2.patch

Hey Ashwanth, thanks for the patch.

I applied it to 0.8 and I want to abuse this ticket to include a bunch of 
trivial "good housekeeping" fixes I think we need for the 0.8 final release 
that probably don't warrant a ticket of thier own.

This includes:
1. Your gc settings, but minus the +AggressiveOpts which doesn't seem to work 
on all versions of java 1.6
2. Clean up the README which has a number of outdated things in it.
3. Remove unused properties from server.properties to fix WARN messages.
4. Change to use log.dirs in the example server config so people know about 
multiple log directories
5. Remove incorrect description of flush.interval in example server config.

> kafka-run-class.sh should use reasonable gc settings
> 
>
> Key: KAFKA-718
> URL: https://issues.apache.org/jira/browse/KAFKA-718
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: 718-v1.patch, KAFKA-718-v2.patch
>
>
> Our start script seems to use the default "stop the world" collector. It 
> would be good to default to well tuned gc settings including gc logging, CMS, 
> etc. Whatever we are using in prod and perf lab...
> Many people who want to use kafka basically don't know java well so they 
> won't succeed in figuring this stuff out on their own and just think it is 
> broken and timing out if we don't have good defaults.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


KAFKA-718

2013-07-11 Thread Jay Kreps
I included a number of "good housekeeping" fixes in with the GC settings
fix here:
https://issues.apache.org/jira/browse/KAFKA-718

I think these should go into 0.8 as they are sloppy, confusing, or in the
case of the GC stuff likely to lead to production problems.

-Jay


Re: Make documentation part of new features acceptance criteria?

2013-07-11 Thread Jay Kreps
Cool, let's do that then.

I think we will likely be in a better state in a week--Sriram is working on
updating the design page which is the last big outstanding thing that is
out of date.

I do totally love the hbase documentation, they are doing it right.

-Jay


On Thu, Jul 11, 2013 at 2:47 AM, Cosmin Lehene  wrote:

> I like the release criteria idea. Perhaps add them to coding guide or the
> developer section on wiki?
>
> WRT feature completeness, I didn't think about having a doc for each one,
> but rather updating the existing doc or the CHANGES.txt file (we don't
> have one yet) with a note when adding new configurations, new interfaces
> or new tools.
> I think should be an awareness thing mostly.
> Kafka's documentation is actually pretty decent, otherwise and the Coding
> Guidelines are great.
>
> I'm not sure if this would work for Kafka or not but you may want to look
> at http://hbase.apache.org/book.html for an example of documentation which
> gets versioned with the code.
>
> Cosmin
>
>
>
>
>
>
> On 7/10/13 7:15 PM, "Jay Kreps"  wrote:
>
> >I like the idea of improving our documentation. Help is very much
> >appreciated in this area (but of course the problem is that the people who
> >experience the holes almost by definition can't fill them in). So even
> >just
> >pointing out areas that aren't covered is really helpful.
> >
> >We are in a sort of awkward stage this week because we have a 0.8 beta
> >release but no detailed docs on its internals.
> >
> >WRT your specific proposals. I don't think we should do the documentation
> >with each feature because I think that tends to lead to a bunch of little
> >documents one for each change. I think we effectively get this out of
> >JIRA+wiki today. This usually serves as a fairly complete design doc +
> >commentary be others. It is pretty hard to get information out of this
> >format for a new user, though.
> >
> >We do version control documentation but we can't physically version
> >control
> >it with the code because the code is in git and Apache only allows SVN as
> >a
> >mechanism for publishing to xxx.apache.org. :-(
> >
> >Instead what about this: we add a new release criteria for documentation
> >completeness. It would be good to formalize the release criteria anyway.
> >Informally they are something like
> >1. Developers think it is feature complete
> >2. Unit tests pass
> >3. Integration/stress tests pass
> >4. Some production usage
> >It would be good to add to this list (5) documentation up-to-date and not
> >do a release without this.
> >
> >It is debatable whether this should apply to beta releases, but probably
> >it
> >should. We can certainly apply it to the final 0.8 release if people are
> >on
> >board.
> >
> >-Jay
> >
> >
> >
> >On Wed, Jul 10, 2013 at 1:17 AM, Cosmin Lehene  wrote:
> >
> >> I'm not sure if there's already a guideline like this, but I wouldn't it
> >> make sense to have it in order to keep documentation in sync with the
> >>code?
> >> Also, having this type of documentation as part of the codebase to allow
> >> proper versioning might be a good idea as well.
> >>
> >> Cosmin
> >>
>
>


Re: Added an "operations" page for 0.8 to site

2013-07-11 Thread Jay Kreps
Ack, thanks! Fixed.

-Jay


On Thu, Jul 11, 2013 at 2:22 PM, Roger Hoover wrote:

> Thank you, Jay.
>
> When talking about flush rates, I think you mean the opposite of what was
> said here:
>
> "However very high application flush rates can lead to high latency when
> the flush does occur."
>
> should be
>
> However very low application flush rates (infrequent flushes) can lead to
> high latency when the flush does occur.
>
> Cheers,
>
> Roger
>
>
> On Thu, Jul 11, 2013 at 12:36 PM, Jay Kreps  wrote:
>
> > Probably it is very incomplete. I copied over much of the content from
> 0.7
> > that is still applicable, but I added the following:
> > 1. A section on data centers and mirroring
> > 2. Removed the cut-and-pasted scala code in the monitoring section and
> just
> > gave a high-level description of what we monitor
> > 3. Added a lot of details about what we have learned about Linux, ext3,
> > write latency, etc.
> >
> > http://kafka.apache.org/08/ops.html
> >
> > Let me know any thoughts on how to improve this.
> >
> > -Jay
> >
>


How to release

2013-07-11 Thread Jay Kreps
Cosmin had a good point about adding documentation to the release steps and
critieria. I don't think we actually have a written set of release criteria
and steps.

Joe, you've done the last few releases I wonder if you could give the steps
you went through?

All any suggestions on the things that should be complete prior to a
release? Let's document this even if it is just a few things so we have
something to work off of going forward.

-Jay


Re: How to release

2013-07-11 Thread Joe Stein
yup yup, i have an old ticket for it already
https://issues.apache.org/jira/browse/KAFKA-394

btw, i submitted 0.8.0-beta1 to maven central yesterday morning.  like
watching the paint to dry for it to show up =8^) will send email once i see
it


On Thu, Jul 11, 2013 at 5:46 PM, Jay Kreps  wrote:

> Cosmin had a good point about adding documentation to the release steps
> and critieria. I don't think we actually have a written set of release
> criteria and steps.
>
> Joe, you've done the last few releases I wonder if you could give the
> steps you went through?
>
> All any suggestions on the things that should be complete prior to a
> release? Let's document this even if it is just a few things so we have
> something to work off of going forward.
>
> -Jay
>



-- 

/*
Joe Stein
http://www.linkedin.com/in/charmalloc
Twitter: @allthingshadoop 
*/


Re: Make documentation part of new features acceptance criteria?

2013-07-11 Thread Sriram Subramanian
I must say that the HBASE site is just awesome! They even have multi
language support for documentation.

On 7/11/13 2:34 PM, "Jay Kreps"  wrote:

>Cool, let's do that then.
>
>I think we will likely be in a better state in a week--Sriram is working
>on
>updating the design page which is the last big outstanding thing that is
>out of date.
>
>I do totally love the hbase documentation, they are doing it right.
>
>-Jay
>
>
>On Thu, Jul 11, 2013 at 2:47 AM, Cosmin Lehene  wrote:
>
>> I like the release criteria idea. Perhaps add them to coding guide or
>>the
>> developer section on wiki?
>>
>> WRT feature completeness, I didn't think about having a doc for each
>>one,
>> but rather updating the existing doc or the CHANGES.txt file (we don't
>> have one yet) with a note when adding new configurations, new interfaces
>> or new tools.
>> I think should be an awareness thing mostly.
>> Kafka's documentation is actually pretty decent, otherwise and the
>>Coding
>> Guidelines are great.
>>
>> I'm not sure if this would work for Kafka or not but you may want to
>>look
>> at http://hbase.apache.org/book.html for an example of documentation
>>which
>> gets versioned with the code.
>>
>> Cosmin
>>
>>
>>
>>
>>
>>
>> On 7/10/13 7:15 PM, "Jay Kreps"  wrote:
>>
>> >I like the idea of improving our documentation. Help is very much
>> >appreciated in this area (but of course the problem is that the people
>>who
>> >experience the holes almost by definition can't fill them in). So even
>> >just
>> >pointing out areas that aren't covered is really helpful.
>> >
>> >We are in a sort of awkward stage this week because we have a 0.8 beta
>> >release but no detailed docs on its internals.
>> >
>> >WRT your specific proposals. I don't think we should do the
>>documentation
>> >with each feature because I think that tends to lead to a bunch of
>>little
>> >documents one for each change. I think we effectively get this out of
>> >JIRA+wiki today. This usually serves as a fairly complete design doc +
>> >commentary be others. It is pretty hard to get information out of this
>> >format for a new user, though.
>> >
>> >We do version control documentation but we can't physically version
>> >control
>> >it with the code because the code is in git and Apache only allows SVN
>>as
>> >a
>> >mechanism for publishing to xxx.apache.org. :-(
>> >
>> >Instead what about this: we add a new release criteria for
>>documentation
>> >completeness. It would be good to formalize the release criteria
>>anyway.
>> >Informally they are something like
>> >1. Developers think it is feature complete
>> >2. Unit tests pass
>> >3. Integration/stress tests pass
>> >4. Some production usage
>> >It would be good to add to this list (5) documentation up-to-date and
>>not
>> >do a release without this.
>> >
>> >It is debatable whether this should apply to beta releases, but
>>probably
>> >it
>> >should. We can certainly apply it to the final 0.8 release if people
>>are
>> >on
>> >board.
>> >
>> >-Jay
>> >
>> >
>> >
>> >On Wed, Jul 10, 2013 at 1:17 AM, Cosmin Lehene 
>>wrote:
>> >
>> >> I'm not sure if there's already a guideline like this, but I
>>wouldn't it
>> >> make sense to have it in order to keep documentation in sync with the
>> >>code?
>> >> Also, having this type of documentation as part of the codebase to
>>allow
>> >> proper versioning might be a good idea as well.
>> >>
>> >> Cosmin
>> >>
>>
>>



[jira] [Commented] (KAFKA-943) Move all configuration key string to constants

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-943:
-

I cleaned up the bad config strings I saw in our example server configs as part 
of KAFKA-718.

> Move all configuration key string to constants
> --
>
> Key: KAFKA-943
> URL: https://issues.apache.org/jira/browse/KAFKA-943
> Project: Kafka
>  Issue Type: Improvement
>  Components: config
>Affects Versions: 0.8
>Reporter: Sam Meder
> Attachments: configConstants.patch
>
>
> The current code base has configuration key strings duplicated all over the 
> place. They show up in the actual *Config classes, a lot of tests, command 
> line utilities and other examples. This makes changes hard and error prone. 
> DRY...
> The attached patch moves these configuration keys to constants and replaces 
> their usage with a reference to the constant. It also cleans up a few old 
> properties and a few misconfigured tests. I've admittedly not written a whole 
> lot of Scala, so there may be some improvements that can be made, in 
> particular I am not sure I chose the best strategy for keys needed by the 
> SyncProducerConfigShared trait (or traits in general).

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-898) Add a KafkaMetricsReporter that wraps Librato's reporter

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-898:
-

Hey Scott, thanks for the patch!

I think it would be better for us to avoid trying to include reporters for each 
possible metrics collection system. This should be easy enough for people to 
add into their deployment, and I am concerned about accumulating dependencies. 
Any objection to that strategy?

> Add a KafkaMetricsReporter that wraps Librato's reporter
> 
>
> Key: KAFKA-898
> URL: https://issues.apache.org/jira/browse/KAFKA-898
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Scott Clasen
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


KAFKA-885

2013-07-11 Thread Jay Kreps
Any objection to this patch for 0.8? We seem to be creating our jar twice
in different places. Seems like it should fall under basic housekeeping
issues:

https://issues.apache.org/jira/browse/KAFKA-885


[jira] [Commented] (KAFKA-877) Still getting kafka.common.NotLeaderForPartitionException

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-877:
-

Is this still happening? If not I want to mark it closed...

> Still getting kafka.common.NotLeaderForPartitionException
> -
>
> Key: KAFKA-877
> URL: https://issues.apache.org/jira/browse/KAFKA-877
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8
> Environment: DEV
>Reporter: BalajiSeshadri
>Priority: Blocker
> Attachments: KAFKA-816.jpg
>
>
> Using the below trunk and i still see error happening.Please let us know if 
> this can be fixed.
> https://github.com/apache/kafka.git
> [2013-04-25 16:47:08,924] WARN 
> [console-consumer-24019_MERD7-21964-1366930009136-8b7f9eb7-leader-finder-thread],
>  Failed to add fetcher for [mytopic,0] to broker 
> id:0,host:MERD7-21964.echostar.com,port:9092 
> (kafka.consumer.ConsumerFetcherManager$$anon$1)
> kafka.common.NotLeaderForPartitionException
> at sun.reflect.GeneratedConstructorAccessor1.newInstance(Unknown 
> Source)
> at 
> sun.reflect.DelegatingConstructorAccessorImpl.newInstance(DelegatingConstructorAccessorImpl.java:45)
> at java.lang.reflect.Constructor.newInstance(Constructor.java:525)
> at java.lang.Class.newInstance0(Class.java:372)
> at java.lang.Class.newInstance(Class.java:325)
> at kafka.common.ErrorMapping$.exceptionFor(ErrorMapping.scala:72)
> at 
> kafka.consumer.SimpleConsumer.earliestOrLatestOffset(SimpleConsumer.scala:163)
> at 
> kafka.consumer.ConsumerFetcherThread.handleOffsetOutOfRange(ConsumerFetcherThread.scala:61)
> at 
> kafka.server.AbstractFetcherThread.addPartition(AbstractFetcherThread.scala:167)
> at 
> kafka.server.AbstractFetcherManager.addFetcher(AbstractFetcherManager.scala:48)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1$$anonfun$doWork$3.apply(ConsumerFetcherManager.scala:79)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1$$anonfun$doWork$3.apply(ConsumerFetcherManager.scala:75)
> at 
> scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:95)
> at 
> scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:95)
> at scala.collection.Iterator$class.foreach(Iterator.scala:772)
> at 
> scala.collection.mutable.HashTable$$anon$1.foreach(HashTable.scala:157)
> at 
> scala.collection.mutable.HashTable$class.foreachEntry(HashTable.scala:190)
> at scala.collection.mutable.HashMap.foreachEntry(HashMap.scala:45)
> at scala.collection.mutable.HashMap.foreach(HashMap.scala:95)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1.doWork(ConsumerFetcherManager.scala:75)
> at kafka.utils.ShutdownableThread.run(ShutdownableThread.scala:51)
> We are evaluating Kafka for our new messaging system and we had tough time 
> running in windows.
> We somehow managed to run 0.8 using cygwin but when we run the console 
> producer/consumer,we are not getting messages from consumer.
> Please help us to fix this issue,this might not be related but its keeping on 
> throwing this error on consumer side. 

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Re: KAFKA-885

2013-07-11 Thread Joe Stein
good housecleaning in my book

+1


On Thu, Jul 11, 2013 at 6:01 PM, Jay Kreps  wrote:

> Any objection to this patch for 0.8? We seem to be creating our jar twice
> in different places. Seems like it should fall under basic housekeeping
> issues:
>
> https://issues.apache.org/jira/browse/KAFKA-885
>



-- 

/*
Joe Stein
http://www.linkedin.com/in/charmalloc
Twitter: @allthingshadoop 
*/


[jira] [Commented] (KAFKA-877) Still getting kafka.common.NotLeaderForPartitionException

2013-07-11 Thread BalajiSeshadri (JIRA)

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

BalajiSeshadri commented on KAFKA-877:
--

Please close it,we will reopen it if it happens again.



> Still getting kafka.common.NotLeaderForPartitionException
> -
>
> Key: KAFKA-877
> URL: https://issues.apache.org/jira/browse/KAFKA-877
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8
> Environment: DEV
>Reporter: BalajiSeshadri
>Priority: Blocker
> Attachments: KAFKA-816.jpg
>
>
> Using the below trunk and i still see error happening.Please let us know if 
> this can be fixed.
> https://github.com/apache/kafka.git
> [2013-04-25 16:47:08,924] WARN 
> [console-consumer-24019_MERD7-21964-1366930009136-8b7f9eb7-leader-finder-thread],
>  Failed to add fetcher for [mytopic,0] to broker 
> id:0,host:MERD7-21964.echostar.com,port:9092 
> (kafka.consumer.ConsumerFetcherManager$$anon$1)
> kafka.common.NotLeaderForPartitionException
> at sun.reflect.GeneratedConstructorAccessor1.newInstance(Unknown 
> Source)
> at 
> sun.reflect.DelegatingConstructorAccessorImpl.newInstance(DelegatingConstructorAccessorImpl.java:45)
> at java.lang.reflect.Constructor.newInstance(Constructor.java:525)
> at java.lang.Class.newInstance0(Class.java:372)
> at java.lang.Class.newInstance(Class.java:325)
> at kafka.common.ErrorMapping$.exceptionFor(ErrorMapping.scala:72)
> at 
> kafka.consumer.SimpleConsumer.earliestOrLatestOffset(SimpleConsumer.scala:163)
> at 
> kafka.consumer.ConsumerFetcherThread.handleOffsetOutOfRange(ConsumerFetcherThread.scala:61)
> at 
> kafka.server.AbstractFetcherThread.addPartition(AbstractFetcherThread.scala:167)
> at 
> kafka.server.AbstractFetcherManager.addFetcher(AbstractFetcherManager.scala:48)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1$$anonfun$doWork$3.apply(ConsumerFetcherManager.scala:79)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1$$anonfun$doWork$3.apply(ConsumerFetcherManager.scala:75)
> at 
> scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:95)
> at 
> scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:95)
> at scala.collection.Iterator$class.foreach(Iterator.scala:772)
> at 
> scala.collection.mutable.HashTable$$anon$1.foreach(HashTable.scala:157)
> at 
> scala.collection.mutable.HashTable$class.foreachEntry(HashTable.scala:190)
> at scala.collection.mutable.HashMap.foreachEntry(HashMap.scala:45)
> at scala.collection.mutable.HashMap.foreach(HashMap.scala:95)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1.doWork(ConsumerFetcherManager.scala:75)
> at kafka.utils.ShutdownableThread.run(ShutdownableThread.scala:51)
> We are evaluating Kafka for our new messaging system and we had tough time 
> running in windows.
> We somehow managed to run 0.8 using cygwin but when we run the console 
> producer/consumer,we are not getting messages from consumer.
> Please help us to fix this issue,this might not be related but its keeping on 
> throwing this error on consumer side. 

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-836) Update quickstart for Kafka 0.8

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-836:


Resolution: Fixed
Status: Resolved  (was: Patch Available)

Done!

> Update quickstart for Kafka 0.8
> ---
>
> Key: KAFKA-836
> URL: https://issues.apache.org/jira/browse/KAFKA-836
> Project: Kafka
>  Issue Type: Sub-task
>Affects Versions: 0.8
>Reporter: Neha Narkhede
>Assignee: Jun Rao
> Attachments: quickstart_08.html
>
>
> Update quickstart to match 0.8 deployment steps

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-717) scala 2.10 build support

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-717:
-

Yeah we can do it in 0.8.1 which should happen fairly quickly after 0.8 but we 
are really trying to lock down 0.8 to critical fixes and this is more of a new 
feature (though obviously desirable). Sorry for the hassle.

> scala 2.10 build support
> 
>
> Key: KAFKA-717
> URL: https://issues.apache.org/jira/browse/KAFKA-717
> Project: Kafka
>  Issue Type: Improvement
>  Components: packaging
>Affects Versions: 0.8
>Reporter: Viktor Taranenko
>  Labels: build
> Attachments: 0001-common-changes-for-2.10.patch, 
> 0001-common-changes-for-2.10.patch, 
> 0001-KAFKA-717-Convert-to-scala-2.10.patch, 
> 0002-java-conversions-changes.patch, 0002-java-conversions-changes.patch, 
> 0003-add-2.9.3.patch, 0003-add-2.9.3.patch, 
> 0004-Fix-cross-compile-of-tests-update-to-2.10.2-and-set-.patch, 
> KAFKA-717-complex.patch, KAFKA-717-simple.patch, kafka_scala_2.10.tar.gz
>
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-705) Controlled shutdown doesn't seem to work on more than one broker in a cluster

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-705:
-

Joel, this is done, no?

> Controlled shutdown doesn't seem to work on more than one broker in a cluster
> -
>
> Key: KAFKA-705
> URL: https://issues.apache.org/jira/browse/KAFKA-705
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8
>Reporter: Neha Narkhede
>Assignee: Joel Koshy
>Priority: Critical
>  Labels: bugs
> Attachments: kafka-705-incremental-v2.patch, kafka-705-v1.patch, 
> shutdown_brokers_eat.py, shutdown-command
>
>
> I wrote a script (attached here) to basically round robin through the brokers 
> in a cluster doing the following 2 operations on each of them -
> 1. Send the controlled shutdown admin command. If it succeeds
> 2. Restart the broker
> What I've observed is that only one broker is able to finish the above 
> successfully the first time around. For the rest of the iterations, no broker 
> is able to shutdown using the admin command and every single time it fails 
> with the error message stating the same number of leaders on every broker. 

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Running 0.8 against 0.7 data and zk

2013-07-11 Thread Jay Kreps
This JIRA has a patch but I thought we already did something like this:
https://issues.apache.org/jira/browse/KAFKA-686

Am I just confused?

-Jay


[jira] [Resolved] (KAFKA-682) java.lang.OutOfMemoryError: Java heap space

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps resolved KAFKA-682.
-

Resolution: Fixed

Marking resolved as we fixed a 0.8 bug that impacted memory and improved the 
default GC settings.

> java.lang.OutOfMemoryError: Java heap space
> ---
>
> Key: KAFKA-682
> URL: https://issues.apache.org/jira/browse/KAFKA-682
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8
> Environment: $ uname -a
> Linux rngadam-think 3.5.0-17-generic #28-Ubuntu SMP Tue Oct 9 19:32:08 UTC 
> 2012 i686 i686 i686 GNU/Linux
> $ java -version
> java version "1.7.0_09"
> OpenJDK Runtime Environment (IcedTea7 2.3.3) (7u9-2.3.3-0ubuntu1~12.04.1)
> OpenJDK Server VM (build 23.2-b09, mixed mode)
>Reporter: Ricky Ng-Adam
> Attachments: java_pid22281.hprof.gz, java_pid22281_Leak_Suspects.zip
>
>
> git pull (commit 32dae955d5e2e2dd45bddb628cb07c874241d856)
> ...build...
> ./sbt update
> ./sbt package
> ...run...
> bin/zookeeper-server-start.sh config/zookeeper.properties
> bin/kafka-server-start.sh config/server.properties
> ...then configured fluentd with kafka plugin...
> gem install fluentd --no-ri --no-rdoc
> gem install fluent-plugin-kafka
> fluentd -c ./fluent/fluent.conf -vv
> ...then flood fluentd with messages inputted from syslog and outputted to 
> kafka.
> results in (after about 1 messages of 1K each in 3s):
> [2013-01-05 02:00:52,087] ERROR Closing socket for /127.0.0.1 because of 
> error (kafka.network.Processor)
> java.lang.OutOfMemoryError: Java heap space
> at 
> kafka.api.ProducerRequest$$anonfun$1$$anonfun$apply$1.apply(ProducerRequest.scala:45)
> at 
> kafka.api.ProducerRequest$$anonfun$1$$anonfun$apply$1.apply(ProducerRequest.scala:42)
> 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.immutable.Range$ByOne$class.foreach(Range.scala:282)
> at scala.collection.immutable.Range$$anon$1.foreach(Range.scala:274)
> at scala.collection.TraversableLike$class.map(TraversableLike.scala:206)
> at scala.collection.immutable.Range.map(Range.scala:39)
> at kafka.api.ProducerRequest$$anonfun$1.apply(ProducerRequest.scala:42)
> at kafka.api.ProducerRequest$$anonfun$1.apply(ProducerRequest.scala:38)
> at 
> scala.collection.TraversableLike$$anonfun$flatMap$1.apply(TraversableLike.scala:227)
> at 
> scala.collection.TraversableLike$$anonfun$flatMap$1.apply(TraversableLike.scala:227)
> at scala.collection.immutable.Range$ByOne$class.foreach(Range.scala:282)
> at scala.collection.immutable.Range$$anon$1.foreach(Range.scala:274)
> at 
> scala.collection.TraversableLike$class.flatMap(TraversableLike.scala:227)
> at scala.collection.immutable.Range.flatMap(Range.scala:39)
> at kafka.api.ProducerRequest$.readFrom(ProducerRequest.scala:38)
> at kafka.api.RequestKeys$$anonfun$1.apply(RequestKeys.scala:32)
> at kafka.api.RequestKeys$$anonfun$1.apply(RequestKeys.scala:32)
> at kafka.network.RequestChannel$Request.(RequestChannel.scala:47)
> at kafka.network.Processor.read(SocketServer.scala:298)
> at kafka.network.Processor.run(SocketServer.scala:209)
> at java.lang.Thread.run(Thread.java:722)

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-649) Cleanup log4j logging

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-649:


Resolution: Fixed
Status: Resolved  (was: Patch Available)

Looks fixed.

> Cleanup log4j logging
> -
>
> Key: KAFKA-649
> URL: https://issues.apache.org/jira/browse/KAFKA-649
> Project: Kafka
>  Issue Type: Improvement
>Affects Versions: 0.8
>Reporter: Jay Kreps
>Assignee: Jun Rao
>Priority: Blocker
> Attachments: kafka-649_extra.patch, kafka-649.patch
>
>
> Review the logs and do the following:
> 1. Fix confusing or duplicative messages
> 2. Assess that messages are at the right level (TRACE/DEBUG/INFO/WARN/ERROR)
> It would also be nice to add a log4j logger for the request logging (i.e. the 
> access log) and another for the controller state change log, since these 
> really have their own use cases.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-677) Retention process gives exception if an empty segment is chosen for collection

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-677:
-

Jun, is this still relevant?

> Retention process gives exception if an empty segment is chosen for collection
> --
>
> Key: KAFKA-677
> URL: https://issues.apache.org/jira/browse/KAFKA-677
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8
>Reporter: Jay Kreps
>Assignee: Jay Kreps
> Fix For: 0.8.1
>
> Attachments: kafka_677-cleanup.patch
>
>
> java.io.FileNotFoundException: 
> /mnt/u001/kafka_08_long_running_test/kafka-logs/NewsActivityEvent-3/.index
>  (No such file or directory)
> at java.io.RandomAccessFile.open(Native Method)
> at java.io.RandomAccessFile.(RandomAccessFile.java:212)
> at kafka.log.OffsetIndex.resize(OffsetIndex.scala:244)
> at kafka.log.OffsetIndex.trimToValidSize(OffsetIndex.scala:233)
> at kafka.log.Log.rollToOffset(Log.scala:459)
> at kafka.log.Log.roll(Log.scala:443)
> at kafka.log.Log.markDeletedWhile(Log.scala:395)
> at 
> kafka.log.LogManager.kafka$log$LogManager$$cleanupExpiredSegments(LogManager.scala:241)
> at 
> kafka.log.LogManager$$anonfun$cleanupLogs$2.apply(LogManager.scala:277)
> at 
> kafka.log.LogManager$$anonfun$cleanupLogs$2.apply(LogManager.scala:275)
> at scala.collection.Iterator$class.foreach(Iterator.scala:631)
> at 
> scala.collection.JavaConversions$JIteratorWrapper.foreach(JavaConversions.scala:474)
> at scala.collection.IterableLike$class.foreach(IterableLike.scala:79)
> at 
> scala.collection.JavaConversions$JCollectionWrapper.foreach(JavaConversions.scala:495)
> at kafka.log.LogManager.cleanupLogs(LogManager.scala:275)
> at 
> kafka.log.LogManager$$anonfun$startup$1.apply$mcV$sp(LogManager.scala:141)
> at kafka.utils.Utils$$anon$2.run(Utils.scala:66)
> at 
> java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:441)
> at 
> java.util.concurrent.FutureTask$Sync.innerRunAndReset(FutureTask.java:317)
> at java.util.concurrent.FutureTask.runAndReset(FutureTask.java:150)
> at 
> java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.access$101(ScheduledThreadPoolExecutor.java:98)
> at 
> java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.runPeriodic(ScheduledThreadPoolExecutor.java:181)
> at 
> java.util.concurrent.ScheduledThreadPoolExecutor$ScheduledFutureTask.run(ScheduledThreadPoolExecutor.java:205)
> at 
> java.util.concurrent.ThreadPoolExecutor$Worker.runTask(ThreadPoolExecutor.java:886)
> at 
> java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:908)
> at java.lang.Thread.run(Thread.java:619)

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-898) Add a KafkaMetricsReporter that wraps Librato's reporter

2013-07-11 Thread Scott Clasen (JIRA)

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

Scott Clasen commented on KAFKA-898:


Hey Jay

   Makes sense. 

I had seen a bunch of questions on the mailing list around "How do I monitor 
Kafka/How do I see metrics from kafka" and figured this would be a super easy 
way for folks to start looking at metrics...and since Coda is an advisor at 
Librato, thought this might be a more 'preferred' reporter...But perhaps 
Librato themselves should publish this to maven central. Once Kafka is 
there...nudge nudge.. :) 

Or alternatively is there a place in contrib that this can live that makes it 
not part of the core dependency chain but easy to use if desired?  

Cheers
SC

> Add a KafkaMetricsReporter that wraps Librato's reporter
> 
>
> Key: KAFKA-898
> URL: https://issues.apache.org/jira/browse/KAFKA-898
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Scott Clasen
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Resolved] (KAFKA-547) The ConsumerStats MBean name should include the groupid

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps resolved KAFKA-547.
-

Resolution: Fixed

At this point I think we should pass on this for 0.8 since we anyway have it in 
trunk.

> The ConsumerStats MBean name should include the groupid
> ---
>
> Key: KAFKA-547
> URL: https://issues.apache.org/jira/browse/KAFKA-547
> Project: Kafka
>  Issue Type: Improvement
>Affects Versions: 0.7.1
>Reporter: Michael Tamm
> Attachments: 
> 0001-consumer-stats-MBean-name-includes-the-group-id-now.patch
>
>
> If there are multiple consumers with different groupids, there should be one 
> mbean per groupid. 

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Resolved] (KAFKA-465) Performance test scripts - refactoring leftovers from tools to perf package

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps resolved KAFKA-465.
-

Resolution: Fixed

As mentioned I think this was fixed in KAFKA-425.

> Performance test scripts - refactoring leftovers from tools to perf package
> ---
>
> Key: KAFKA-465
> URL: https://issues.apache.org/jira/browse/KAFKA-465
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8
>Reporter: Pierre Queinnec
>  Labels: patch
> Attachments: KAFKA-465.diff
>
>
> The performance test shell scripts seem like they weren't updated to the new 
> package hierarchy, and still reference kafka.tools.ProducerPerformance for 
> example. Patch attached.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-397) kafka.common.InvalidMessageSizeException: null

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-397:
-

So is there anything we should be fixing here?

> kafka.common.InvalidMessageSizeException: null
> --
>
> Key: KAFKA-397
> URL: https://issues.apache.org/jira/browse/KAFKA-397
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.7.1
>Reporter: David Siegel
>Priority: Blocker
> Attachments: kafka-logs-invalid-message-size-null.tar
>
>
> I've just gotten the following error while running the zookeeper consumer.  
> I made a backup of the kafka log directory and wiped the logs.  I restarting 
> kafka and the consumer.  After processing a few hundred messages successfully 
> I got the same error again.  I restarted the consumer again and got the same 
> error immediately.  I wiped the logs yet again and reproduced the error 
> again.  I will attach the logs from this final run.
> I'm running Kafka 0.7.1
> I have the following message size configurations:
>   producer config:
> max.message.size: 100
>   consumer config:
> fetch.size: 2072000
> Does it matter that I'm only wiping the logs and not wiping the zookeeper 
> offsets?
> 2012-07-10 02:31:21,998 ERROR [Consumer1] 
> c.k.h.c.k.KafkaConsumerServiceWorker: Failed to get next student event
> kafka.common.InvalidMessageSizeException: null
> at sun.reflect.NativeConstructorAccessorImpl.newInstance0(Native 
> Method) ~[na:1.6.0_30]
> at 
> sun.reflect.NativeConstructorAccessorImpl.newInstance(NativeConstructorAccessorImpl.java:39)
>  ~[na:1.6.0_30]
> at 
> sun.reflect.DelegatingConstructorAccessorImpl.newInstance(DelegatingConstructorAccessorImpl.java:27)
>  ~[na:1.6.0_30]
> at java.lang.reflect.Constructor.newInstance(Constructor.java:513) 
> ~[na:1.6.0_30]
> at java.lang.Class.newInstance0(Class.java:355) ~[na:1.6.0_30]
> at java.lang.Class.newInstance(Class.java:308) ~[na:1.6.0_30]
> at 
> kafka.common.ErrorMapping$.maybeThrowException(ErrorMapping.scala:53) 
> ~[KPIP-0.4.birdy.jar:na]
> at 
> kafka.message.ByteBufferMessageSet.kafka$message$ByteBufferMessageSet$$internalIterator(ByteBufferMessageSet.scala:99)
>  ~[KPIP-0.4.birdy.jar:na]
> at 
> kafka.message.ByteBufferMessageSet.iterator(ByteBufferMessageSet.scala:82) 
> ~[KPIP-0.4.birdy.jar:na]
> at 
> kafka.consumer.ConsumerIterator.makeNext(ConsumerIterator.scala:81) 
> ~[KPIP-0.4.birdy.jar:na]
> at 
> kafka.consumer.ConsumerIterator.makeNext(ConsumerIterator.scala:32) 
> ~[KPIP-0.4.birdy.jar:na]
> at 
> kafka.utils.IteratorTemplate.maybeComputeNext(IteratorTemplate.scala:59) 
> ~[KPIP-0.4.birdy.jar:na]
> at kafka.utils.IteratorTemplate.hasNext(IteratorTemplate.scala:51) 
> ~[KPIP-0.4.birdy.jar:na]
> at kafka.utils.IteratorTemplate.next(IteratorTemplate.scala:36) 
> ~[KPIP-0.4.birdy.jar:na]
> at kafka.consumer.ConsumerIterator.next(ConsumerIterator.scala:43) 
> ~[KPIP-0.4.birdy.jar:na]
> at java.lang.Thread.run(Thread.java:662) [na:1.6.0_30]
> 2012-07-10 02:31:21,998 ERROR [Consumer1] 
> c.k.h.c.k.KafkaConsumerServiceWorker: Iterator got into bad state.  Thread 
> exiting
> java.lang.IllegalStateException: Iterator is in failed state
> at kafka.utils.IteratorTemplate.hasNext(IteratorTemplate.scala:47) 
> ~[KPIP-0.4.birdy.jar:na]
> at kafka.utils.IteratorTemplate.next(IteratorTemplate.scala:36) 
> ~[KPIP-0.4.birdy.jar:na]
> at kafka.consumer.ConsumerIterator.next(ConsumerIterator.scala:43) 
> ~[KPIP-0.4.birdy.jar:na]
> at java.lang.Thread.run(Thread.java:662) [na:1.6.0_30]

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Resolved] (KAFKA-399) 0.7.1 seems to show less performance than 0.7.0

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps resolved KAFKA-399.
-

Resolution: Won't Fix

0.8 is different enough that I think it is safe to mark this "won't fix".

> 0.7.1 seems to show less performance than 0.7.0
> ---
>
> Key: KAFKA-399
> URL: https://issues.apache.org/jira/browse/KAFKA-399
> Project: Kafka
>  Issue Type: Test
>  Components: core
>Affects Versions: 0.7, 0.7.1
>Reporter: Dongmin Yu
>Priority: Minor
> Attachments: DirSource.java, dstat.txt
>
>
> On a test, 0.7.1 seems to show less performance than 0.7.0.
> Producer is on a machine A, and Broker is on machine B.
> Machine's spec is
> - 8 core Intel(R) Xeon(R) CPU   E5405  @ 2.00GHz
> - 16G RAM
> Broker's configuration is
> - num.threads=8
> - socket.send.buffer=1048576
> - socket.receive.buffer=1048576
> - max.socket.request.bytes=104857600
> - log.flush.interval=1
> - log.default.flush.interval.ms=1000
> - log.default.flush.scheduler.interval.ms=1000
> - log.file.size=536870912
> - enable.zookeeper=true
> Additional note
> - no compression
> Attached dstat result and used producer code

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-717) scala 2.10 build support

2013-07-11 Thread Matei Zaharia (JIRA)

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

Matei Zaharia commented on KAFKA-717:
-

Sounds good! Glad to hear you guys are looking at it.

> scala 2.10 build support
> 
>
> Key: KAFKA-717
> URL: https://issues.apache.org/jira/browse/KAFKA-717
> Project: Kafka
>  Issue Type: Improvement
>  Components: packaging
>Affects Versions: 0.8
>Reporter: Viktor Taranenko
>  Labels: build
> Attachments: 0001-common-changes-for-2.10.patch, 
> 0001-common-changes-for-2.10.patch, 
> 0001-KAFKA-717-Convert-to-scala-2.10.patch, 
> 0002-java-conversions-changes.patch, 0002-java-conversions-changes.patch, 
> 0003-add-2.9.3.patch, 0003-add-2.9.3.patch, 
> 0004-Fix-cross-compile-of-tests-update-to-2.10.2-and-set-.patch, 
> KAFKA-717-complex.patch, KAFKA-717-simple.patch, kafka_scala_2.10.tar.gz
>
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-318) update zookeeper dependency to 3.3.5

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-318:
-

I recommend we just take this on trunk. It will get in 0.8.1 soon enough.

> update zookeeper dependency to 3.3.5
> 
>
> Key: KAFKA-318
> URL: https://issues.apache.org/jira/browse/KAFKA-318
> Project: Kafka
>  Issue Type: Improvement
>  Components: core
>Reporter: Pierre-Yves Ritschard
> Attachments: 
> 0001-upgrade-zookeeper-to-3.3.5-since-it-fixes-a-bunch-of.patch
>
>
> zookeeper 3.3.5 fixes a bunch of nasty bugs, update from 3.3.4

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-852) Remove clientId from OffsetFetchResponse and OffsetCommitResponse

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-852:


Resolution: Fixed
Status: Resolved  (was: Patch Available)

Sorry for lagging on this. Committed.

> Remove clientId from OffsetFetchResponse and OffsetCommitResponse
> -
>
> Key: KAFKA-852
> URL: https://issues.apache.org/jira/browse/KAFKA-852
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8.1
>Reporter: David Arthur
>Assignee: David Arthur
>Priority: Minor
> Fix For: 0.8.1
>
> Attachments: 
> 0001-KAFKA-852-remove-clientId-from-Offset-Fetch-Commit-R.patch, 
> KAFKA-852.diff, KAFKA-852v2.diff
>
>   Original Estimate: 2h
>  Remaining Estimate: 2h
>
> These are not needed and conflict with the API documentation. Should be 
> removed to be consistent with other APIs

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Resolved] (KAFKA-278) Issues partitioning a new topic

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps resolved KAFKA-278.
-

Resolution: Fixed

This should be fixed in 0.8 where we handle partitioning properly.

> Issues partitioning a new topic
> ---
>
> Key: KAFKA-278
> URL: https://issues.apache.org/jira/browse/KAFKA-278
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.7
>Reporter: Matt
>Priority: Minor
> Attachments: bootstrap_new_brokers.patch
>
>
> There are two cases where correct partitioning fails for a new topic.
> Case 1: Topic exists on current Kafka cluster. A new broker is added to the 
> cluster. The new broker will never host partitions for the existing topic.
> To reproduce:
> 1) Create a cluster of brokers along with a ZooKeeper ensemble.
> 2) Send messages for a topic to the cluster.
> 3) Add a new broker to the cluster.
> 4) New broker will never see the existing topic.
> Case 2: Topic does not exist on current Kafka cluster. Producer sends 
> messages to a new topic that did not previously exist in the cluster. If, 
> during the producer session, one or more partitions are not created on a 
> broker, the broker will never host those partitions.
> To reproduce:
> 1) Create a cluster of brokers along with a ZooKeeper ensemble.
> 2) Send messages to a new topic.
> 3) Shut down the producer before the topic is created on at least one broker.
> 4) The broker that did not allocate the topic will never host the topic.
> My guess(!) here is that when a new producer is created, it gets a list of 
> topics and partitions based on the current state of the brokers in the 
> cluster. Since some brokers are missing the topic, the producer will never 
> send messages to that broker and partitions will never be created.
> Work around:
> Manually create the topic/partition directories in the kafka logs directory 
> and reboot kafka. It will register the topic/partitions in ZooKeeper.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-273) Occassional GZIP errors on the server while writing compressed data to disk

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-273:
-

Is this still happening?

> Occassional GZIP errors on the server while writing compressed data to disk
> ---
>
> Key: KAFKA-273
> URL: https://issues.apache.org/jira/browse/KAFKA-273
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.7
>Reporter: Neha Narkhede
>Assignee: Neha Narkhede
> Attachments: kafka-273.patch
>
>
> Occasionally, we see the following errors on the Kafka server -
> 2012/02/08 14:58:21.832 ERROR [KafkaRequestHandlers] [kafka-processor-6] 
> [kafka] Error processing MultiProducerRequest on NusImpressionSetEvent:0
> java.io.EOFException: Unexpected end of ZLIB input stream
> at 
> java.util.zip.InflaterInputStream.fill(InflaterInputStream.java:223)
> at 
> java.util.zip.InflaterInputStream.read(InflaterInputStream.java:141)
> at java.util.zip.GZIPInputStream.read(GZIPInputStream.java:92)
> at java.io.FilterInputStream.read(FilterInputStream.java:90)
> at kafka.message.GZIPCompression.read(CompressionUtils.scala:52)
> at 
> kafka.message.CompressionUtils$$anonfun$decompress$1.apply$mcI$sp(CompressionUtils.scala:143)
> at 
> kafka.message.CompressionUtils$$anonfun$decompress$1.apply(CompressionUtils.scala:143)
> at 
> kafka.message.CompressionUtils$$anonfun$decompress$1.apply(CompressionUtils.scala:143)
> at 
> scala.collection.immutable.Stream$$anonfun$continually$1.apply(Stream.scala:598)
> at 
> scala.collection.immutable.Stream$$anonfun$continually$1.apply(Stream.scala:598)
> at scala.collection.immutable.Stream$Cons.tail(Stream.scala:555)
> at scala.collection.immutable.Stream$Cons.tail(Stream.scala:549)
> at 
> scala.collection.immutable.Stream$$anonfun$takeWhile$1.apply(Stream.scala:394)
> at 
> scala.collection.immutable.Stream$$anonfun$takeWhile$1.apply(Stream.scala:394)
> at scala.collection.immutable.Stream$Cons.tail(Stream.scala:555)
> at scala.collection.immutable.Stream$Cons.tail(Stream.scala:549)
> at scala.collection.immutable.Stream.foreach(Stream.scala:255)
> at 
> kafka.message.CompressionUtils$.decompress(CompressionUtils.scala:143)
> at 
> kafka.message.ByteBufferMessageSet$$anon$1.makeNextOuter(ByteBufferMessageSet.scala:119)
> at 
> kafka.message.ByteBufferMessageSet$$anon$1.makeNext(ByteBufferMessageSet.scala:132)
> at 
> kafka.message.ByteBufferMessageSet$$anon$1.makeNext(ByteBufferMessageSet.scala:81)
> at 
> kafka.utils.IteratorTemplate.maybeComputeNext(IteratorTemplate.scala:59)
> at kafka.utils.IteratorTemplate.hasNext(IteratorTemplate.scala:51)
> at scala.collection.Iterator$class.foreach(Iterator.scala:631)
> at kafka.utils.IteratorTemplate.foreach(IteratorTemplate.scala:30)
> at scala.collection.IterableLike$class.foreach(IterableLike.scala:79)
> at kafka.message.MessageSet.foreach(MessageSet.scala:87)
> at kafka.log.Log.append(Log.scala:204)
> at 
> kafka.server.KafkaRequestHandlers.kafka$server$KafkaRequestHandlers$$handleProducerRequest(KafkaRequestHandlers.scala:70)
> at 
> kafka.server.KafkaRequestHandlers$$anonfun$handleMultiProducerRequest$1.apply(KafkaRequestHandlers.scala:63)
> at 
> kafka.server.KafkaRequestHandlers$$anonfun$handleMultiProducerRequest$1.apply(KafkaRequestHandlers.scala:63)
> 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.IndexedSeqOptimized$class.foreach(IndexedSeqOptimized.scala:34)
> at scala.collection.mutable.ArrayOps.foreach(ArrayOps.scala:34)
> at 
> scala.collection.TraversableLike$class.map(TraversableLike.scala:206)
> at scala.collection.mutable.ArrayOps.map(ArrayOps.scala:34)
> at 
> kafka.server.KafkaRequestHandlers.handleMultiProducerRequest(KafkaRequestHandlers.scala:63)
> at 
> kafka.server.KafkaRequestHandlers$$anonfun$handlerFor$4.apply(KafkaRequestHandlers.scala:42)
> at 
> kafka.server.KafkaRequestHandlers$$anonfun$handlerFor$4.apply(KafkaRequestHandlers.scala:42)
> at kafka.network.Processor.handle(SocketServer.scala:297)
> at kafka.network.Processor.read(SocketServer.scala:320)
> at kafka.network.Processor.run(SocketServer.scala:215)
> at java.lang.Thread.run(Thread.java:619)

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please

[jira] [Commented] (KAFKA-242) Subsequent calls of ConsumerConnector.createMessageStreams cause Consumer offset to be incorrect

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-242:
-

Does this bug still exist in 0.8?

> Subsequent calls of ConsumerConnector.createMessageStreams cause Consumer 
> offset to be incorrect
> 
>
> Key: KAFKA-242
> URL: https://issues.apache.org/jira/browse/KAFKA-242
> Project: Kafka
>  Issue Type: Bug
>  Components: clients
>Affects Versions: 0.7
>Reporter: David Arthur
> Attachments: kafka.log
>
>
> When calling ConsumerConnector.createMessageStreams in rapid succession, the 
> Consumer offset is incorrectly advanced causing the consumer to lose 
> messages. This seems to happen when createMessageStreams is called before the 
> rebalancing triggered by the previous call to createMessageStreams has 
> completed. 

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Re: How to release

2013-07-11 Thread Jay Kreps
Oh, cool.

-Jay


On Thu, Jul 11, 2013 at 2:50 PM, Joe Stein  wrote:

> yup yup, i have an old ticket for it already
> https://issues.apache.org/jira/browse/KAFKA-394
>
> btw, i submitted 0.8.0-beta1 to maven central yesterday morning.  like
> watching the paint to dry for it to show up =8^) will send email once i see
> it
>
>
> On Thu, Jul 11, 2013 at 5:46 PM, Jay Kreps  wrote:
>
>> Cosmin had a good point about adding documentation to the release steps
>> and critieria. I don't think we actually have a written set of release
>> criteria and steps.
>>
>> Joe, you've done the last few releases I wonder if you could give the
>> steps you went through?
>>
>> All any suggestions on the things that should be complete prior to a
>> release? Let's document this even if it is just a few things so we have
>> something to work off of going forward.
>>
>> -Jay
>>
>
>
>
> --
>
> /*
> Joe Stein
> http://www.linkedin.com/in/charmalloc
> Twitter: @allthingshadoop 
> */
>


[jira] [Resolved] (KAFKA-877) Still getting kafka.common.NotLeaderForPartitionException

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps resolved KAFKA-877.
-

Resolution: Cannot Reproduce

Will reopen if it reoccurs.

> Still getting kafka.common.NotLeaderForPartitionException
> -
>
> Key: KAFKA-877
> URL: https://issues.apache.org/jira/browse/KAFKA-877
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8
> Environment: DEV
>Reporter: BalajiSeshadri
>Priority: Blocker
> Attachments: KAFKA-816.jpg
>
>
> Using the below trunk and i still see error happening.Please let us know if 
> this can be fixed.
> https://github.com/apache/kafka.git
> [2013-04-25 16:47:08,924] WARN 
> [console-consumer-24019_MERD7-21964-1366930009136-8b7f9eb7-leader-finder-thread],
>  Failed to add fetcher for [mytopic,0] to broker 
> id:0,host:MERD7-21964.echostar.com,port:9092 
> (kafka.consumer.ConsumerFetcherManager$$anon$1)
> kafka.common.NotLeaderForPartitionException
> at sun.reflect.GeneratedConstructorAccessor1.newInstance(Unknown 
> Source)
> at 
> sun.reflect.DelegatingConstructorAccessorImpl.newInstance(DelegatingConstructorAccessorImpl.java:45)
> at java.lang.reflect.Constructor.newInstance(Constructor.java:525)
> at java.lang.Class.newInstance0(Class.java:372)
> at java.lang.Class.newInstance(Class.java:325)
> at kafka.common.ErrorMapping$.exceptionFor(ErrorMapping.scala:72)
> at 
> kafka.consumer.SimpleConsumer.earliestOrLatestOffset(SimpleConsumer.scala:163)
> at 
> kafka.consumer.ConsumerFetcherThread.handleOffsetOutOfRange(ConsumerFetcherThread.scala:61)
> at 
> kafka.server.AbstractFetcherThread.addPartition(AbstractFetcherThread.scala:167)
> at 
> kafka.server.AbstractFetcherManager.addFetcher(AbstractFetcherManager.scala:48)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1$$anonfun$doWork$3.apply(ConsumerFetcherManager.scala:79)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1$$anonfun$doWork$3.apply(ConsumerFetcherManager.scala:75)
> at 
> scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:95)
> at 
> scala.collection.mutable.HashMap$$anonfun$foreach$1.apply(HashMap.scala:95)
> at scala.collection.Iterator$class.foreach(Iterator.scala:772)
> at 
> scala.collection.mutable.HashTable$$anon$1.foreach(HashTable.scala:157)
> at 
> scala.collection.mutable.HashTable$class.foreachEntry(HashTable.scala:190)
> at scala.collection.mutable.HashMap.foreachEntry(HashMap.scala:45)
> at scala.collection.mutable.HashMap.foreach(HashMap.scala:95)
> at 
> kafka.consumer.ConsumerFetcherManager$$anon$1.doWork(ConsumerFetcherManager.scala:75)
> at kafka.utils.ShutdownableThread.run(ShutdownableThread.scala:51)
> We are evaluating Kafka for our new messaging system and we had tough time 
> running in windows.
> We somehow managed to run 0.8 using cygwin but when we run the console 
> producer/consumer,we are not getting messages from consumer.
> Please help us to fix this issue,this might not be related but its keeping on 
> throwing this error on consumer side. 

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-898) Add a KafkaMetricsReporter that wraps Librato's reporter

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-898:
-

We can dump it in contrib, but I think what we have found is that contrib is a 
pretty bad place for plugins. It takes less time to make a github project then 
to interact on JIRA and that way the author can keep iterating on the code 
unencumbered. So I think the thinking on contrib is to instead either take it 
as a first-class part of the project that we maintain indefinitely or let it 
thrive elsewhere.

> Add a KafkaMetricsReporter that wraps Librato's reporter
> 
>
> Key: KAFKA-898
> URL: https://issues.apache.org/jira/browse/KAFKA-898
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Scott Clasen
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-898) Add a KafkaMetricsReporter that wraps Librato's reporter

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-898:


Resolution: Won't Fix
Status: Resolved  (was: Patch Available)

Will be happier elsewhere.

> Add a KafkaMetricsReporter that wraps Librato's reporter
> 
>
> Key: KAFKA-898
> URL: https://issues.apache.org/jira/browse/KAFKA-898
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Scott Clasen
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Re: having problem with 0.8 gzip compression

2013-07-11 Thread Scott Wang
Ok, the problem solved, I think it might be because some of the jar files
that I was using were "OLD".  I was building the producer and consumer
under the 0.7 environment except swapping out the kafka jar file.   Now, I
created a whole new environment and pull in all the jar files from the
0.8.  That seems to solve my 0.8 gzip problem.   Thank you for all the
help.


[jira] [Commented] (KAFKA-898) Add a KafkaMetricsReporter that wraps Librato's reporter

2013-07-11 Thread Scott Clasen (JIRA)

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

Scott Clasen commented on KAFKA-898:


Cool ill see if the librato folks want it, if not ill publish something to 
maven central, once Kafka is there

> Add a KafkaMetricsReporter that wraps Librato's reporter
> 
>
> Key: KAFKA-898
> URL: https://issues.apache.org/jira/browse/KAFKA-898
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Scott Clasen
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Closed] (KAFKA-898) Add a KafkaMetricsReporter that wraps Librato's reporter

2013-07-11 Thread Scott Clasen (JIRA)

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

Scott Clasen closed KAFKA-898.
--


> Add a KafkaMetricsReporter that wraps Librato's reporter
> 
>
> Key: KAFKA-898
> URL: https://issues.apache.org/jira/browse/KAFKA-898
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Scott Clasen
>


--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-718) kafka-run-class.sh should use reasonable gc settings

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-718:


Attachment: KAFKA-718-v3.patch

Separate out a few more command line variables (GC, GC logging, SCALA_VERSION, 
etc).

Also remove delete_topic.sh command since that doesn't work yet.

> kafka-run-class.sh should use reasonable gc settings
> 
>
> Key: KAFKA-718
> URL: https://issues.apache.org/jira/browse/KAFKA-718
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: 718-v1.patch, KAFKA-718-v2.patch, KAFKA-718-v3.patch
>
>
> Our start script seems to use the default "stop the world" collector. It 
> would be good to default to well tuned gc settings including gc logging, CMS, 
> etc. Whatever we are using in prod and perf lab...
> Many people who want to use kafka basically don't know java well so they 
> won't succeed in figuring this stuff out on their own and just think it is 
> broken and timing out if we don't have good defaults.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-637) Separate log4j environment variable from KAFKA_OPTS in kafka-run-class.sh

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps updated KAFKA-637:


Resolution: Fixed
  Assignee: Jay Kreps
Status: Resolved  (was: Patch Available)

This is sensible. Merged this in with KAFKA-718 which has a few other command 
line cleanups. Would you mind taking a look at that and let's continue any 
discussion there.

> Separate log4j environment variable from KAFKA_OPTS in kafka-run-class.sh
> -
>
> Key: KAFKA-637
> URL: https://issues.apache.org/jira/browse/KAFKA-637
> Project: Kafka
>  Issue Type: Improvement
>Affects Versions: 0.8.1
>Reporter: Chris Curtin
>Assignee: Jay Kreps
>Priority: Minor
> Attachments: KAFKA-637.patch
>
>
> The kafka-run-class.sh script has an environment variable for 'KAFKA_OPTS' 
> which defines the log4j configuration file with other system variables.
> By moving the log4j settings into their own variable we can set that 
> configuration file without having to also include the system parameters. This 
> way if the script changes in the next release our management scripts won't 
> have (potentially) unexpected side effects.
> We change the logging configuration file in the test environment so we can 
> run multiple brokers on a single machine for testing. Without this option all 
> the logs for all the brokers get jumbled together.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-718) kafka-run-class.sh should use reasonable gc settings

2013-07-11 Thread Ashwanth Fernando (JIRA)

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

Ashwanth Fernando commented on KAFKA-718:
-

Thanks [~jkreps]!! Can we mark this as resolved or do you have some more?

> kafka-run-class.sh should use reasonable gc settings
> 
>
> Key: KAFKA-718
> URL: https://issues.apache.org/jira/browse/KAFKA-718
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: 718-v1.patch, KAFKA-718-v2.patch, KAFKA-718-v3.patch
>
>
> Our start script seems to use the default "stop the world" collector. It 
> would be good to default to well tuned gc settings including gc logging, CMS, 
> etc. Whatever we are using in prod and perf lab...
> Many people who want to use kafka basically don't know java well so they 
> won't succeed in figuring this stuff out on their own and just think it is 
> broken and timing out if we don't have good defaults.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Comment Edited] (KAFKA-718) kafka-run-class.sh should use reasonable gc settings

2013-07-11 Thread Ashwanth Fernando (JIRA)

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

Ashwanth Fernando edited comment on KAFKA-718 at 7/11/13 11:40 PM:
---

Thanks [~jkreps] !! Can we mark this as resolved or do you have some more?

  was (Author: ashwanthferna...@gmail.com):
Thanks [~jkreps]!! Can we mark this as resolved or do you have some more?
  
> kafka-run-class.sh should use reasonable gc settings
> 
>
> Key: KAFKA-718
> URL: https://issues.apache.org/jira/browse/KAFKA-718
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: 718-v1.patch, KAFKA-718-v2.patch, KAFKA-718-v3.patch
>
>
> Our start script seems to use the default "stop the world" collector. It 
> would be good to default to well tuned gc settings including gc logging, CMS, 
> etc. Whatever we are using in prod and perf lab...
> Many people who want to use kafka basically don't know java well so they 
> won't succeed in figuring this stuff out on their own and just think it is 
> broken and timing out if we don't have good defaults.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-946) Kafka Hadoop Consumer fails when verifying message checksum

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-946:
-

>From Richard:

Sorry, It took a while to remember the context.
In a Kafka Message, the checksum is created on the whole message: header and 
payload included.

The contrib code passes only the Message payload to the mapper, and not the 
whole buffer. I believe the reason for this is that we wanted to pass just the 
message data (not any of the kafka special bits) for the mapper handle. The 
Message that is created in the SimpleKafkaETLMapper is then creating using the 
incorrect payload bytes. It can be argued that this is desirable. For instance, 
Mappers can decode the byte buffer directly into Avro without stripping away 
the header or dealing with kafka Messages at all.

Also, changing the KafakETLContext code could be affect a lot of users. This is 
definitely not a backwards compatible change. It can also be argued that the 
BytesWriteable only contains the payload code, and that checksum-ing of the 
message should've occurred well before the Mapper gets the message.

However, I think that Sam's fix still has merit. It would be good for the 
KafkaETLContext to pass the Message buffer instead of the payload and the 
RecordReader could strip away the kafka bits before giving the payload to the 
Mapper. Perhaps put in a config switch to either get just payload or the whole 
kafka message buffer?

Additional thoughts:
I assume there are plenty of users of this code. If there's anyone who uses the 
KafkaETLContext directly, they'll find the patch's changes to break their 
stuff. However, for those who are using KafkaETLContext through the 
KafkaETLRecordReader (as they should), then there is a way to make it backwards 
compatible.

The checksumming and payload stripping code could go into the RecordReader 
rather than the KafkaETLContext.

If the scope of these changes are too big, I'd just fix the 
SimpleKafkaETLMapper to not parse the payload bytes.

> Kafka Hadoop Consumer fails when verifying message checksum
> ---
>
> Key: KAFKA-946
> URL: https://issues.apache.org/jira/browse/KAFKA-946
> Project: Kafka
>  Issue Type: Bug
>  Components: contrib
>Affects Versions: 0.8
>Reporter: Sam Meder
>Priority: Critical
> Attachments: hadoop_consumer.patch
>
>
> The code tries to verify the checksum, but fails because the data available 
> isn't the same. In KafkaETLContext:
> protected boolean get(KafkaETLKey key, BytesWritable value) throws 
> IOException {
>   if (_messageIt != null && _messageIt.hasNext()) {
> MessageAndOffset messageAndOffset = _messageIt.next();
> ByteBuffer buf = messageAndOffset.message().payload();
> int origSize = buf.remaining();
> byte[] bytes = new byte[origSize];
>   buf.get(bytes, buf.position(), origSize);
> value.set(bytes, 0, origSize);
> key.set(_index, _offset, messageAndOffset.message().checksum());
> _offset = messageAndOffset.nextOffset();  //increase offset   
>   
>  
> _count ++;  //increase count  
>   
>  
> return true;
> }
> else return false;
> }
> Note that the message payload is used and the message checksum is included in 
> the key. The in SimpleKafkaETLMapper:
> @Override
> public void map(KafkaETLKey key, BytesWritable val,
> OutputCollector collector,
> Reporter reporter) throws IOException {
>   byte[] bytes = KafkaETLUtils.getBytes(val);
> //check the checksum of message   
>   
>  
> Message message = new Message(bytes);
> long checksum = key.getChecksum();
>   if (checksum != message.checksum())
> throw new IOException ("Invalid message checksum "
> + message.checksum() + ". 
> Expected " + key + ".");
> the Message object is initialized with the payload bytes and a new checksum 
> is calculated. The problem is that the original message checksum also 
> contains the key so checksum verification fails...

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see

[jira] [Comment Edited] (KAFKA-718) kafka-run-class.sh should use reasonable gc settings

2013-07-11 Thread Ashwanth Fernando (JIRA)

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

Ashwanth Fernando edited comment on KAFKA-718 at 7/11/13 11:40 PM:
---

Thanks Jay !! Can we mark this as resolved or do you have some more?

  was (Author: ashwanthferna...@gmail.com):
Thanks [~jkreps] !! Can we mark this as resolved or do you have some more?
  
> kafka-run-class.sh should use reasonable gc settings
> 
>
> Key: KAFKA-718
> URL: https://issues.apache.org/jira/browse/KAFKA-718
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: 718-v1.patch, KAFKA-718-v2.patch, KAFKA-718-v3.patch
>
>
> Our start script seems to use the default "stop the world" collector. It 
> would be good to default to well tuned gc settings including gc logging, CMS, 
> etc. Whatever we are using in prod and perf lab...
> Many people who want to use kafka basically don't know java well so they 
> won't succeed in figuring this stuff out on their own and just think it is 
> broken and timing out if we don't have good defaults.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-718) kafka-run-class.sh should use reasonable gc settings

2013-07-11 Thread Jay Kreps (JIRA)

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

Jay Kreps commented on KAFKA-718:
-

Since 0.8 is somewhat locked down I am just waiting on close sanity checking 
from a few others before I check in. I'll close it up then.

> kafka-run-class.sh should use reasonable gc settings
> 
>
> Key: KAFKA-718
> URL: https://issues.apache.org/jira/browse/KAFKA-718
> Project: Kafka
>  Issue Type: Bug
>Reporter: Jay Kreps
>Assignee: Neha Narkhede
> Attachments: 718-v1.patch, KAFKA-718-v2.patch, KAFKA-718-v3.patch
>
>
> Our start script seems to use the default "stop the world" collector. It 
> would be good to default to well tuned gc settings including gc logging, CMS, 
> etc. Whatever we are using in prod and perf lab...
> Many people who want to use kafka basically don't know java well so they 
> won't succeed in figuring this stuff out on their own and just think it is 
> broken and timing out if we don't have good defaults.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Re: Added an "operations" page for 0.8 to site

2013-07-11 Thread Roger Hoover
Thank you, Jay.

When talking about flush rates, I think you mean the opposite of what was
said here:

"However very high application flush rates can lead to high latency when
the flush does occur."

should be

However very low application flush rates (infrequent flushes) can lead to
high latency when the flush does occur.

Cheers,

Roger


On Thu, Jul 11, 2013 at 12:36 PM, Jay Kreps  wrote:

> Probably it is very incomplete. I copied over much of the content from 0.7
> that is still applicable, but I added the following:
> 1. A section on data centers and mirroring
> 2. Removed the cut-and-pasted scala code in the monitoring section and just
> gave a high-level description of what we monitor
> 3. Added a lot of details about what we have learned about Linux, ext3,
> write latency, etc.
>
> http://kafka.apache.org/08/ops.html
>
> Let me know any thoughts on how to improve this.
>
> -Jay
>


KAFKA-718

2013-07-11 Thread Ashwanth Fernando
Hi Kafka team,
I submitted the patch. Can you let me know how to take this forward?

Thanks,
Ashwanth


[jira] [Updated] (KAFKA-957) MirrorMaker needs to preserve the key in the source cluster

2013-07-11 Thread Guozhang Wang (JIRA)

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

Guozhang Wang updated KAFKA-957:


Attachment: KAFKA-957.v1.patch

4. Add a ByteArrayPartitioner since Array[Byte].hashCode will result in 
different values for objects even with the same content. Enforce MirrorMaker to 
use this ByteArray Partitioner

Passed MirrorMaker testcase 5001

> MirrorMaker needs to preserve the key in the source cluster
> ---
>
> Key: KAFKA-957
> URL: https://issues.apache.org/jira/browse/KAFKA-957
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8
>Reporter: Jun Rao
>Assignee: Guozhang Wang
> Attachments: KAFKA-957.v1.patch
>
>
> Currently, MirrorMaker only propagates the message to the target cluster, but 
> not the associated key.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Updated] (KAFKA-957) MirrorMaker needs to preserve the key in the source cluster

2013-07-11 Thread Guozhang Wang (JIRA)

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

Guozhang Wang updated KAFKA-957:


Status: Patch Available  (was: Open)

> MirrorMaker needs to preserve the key in the source cluster
> ---
>
> Key: KAFKA-957
> URL: https://issues.apache.org/jira/browse/KAFKA-957
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8
>Reporter: Jun Rao
>Assignee: Guozhang Wang
> Attachments: KAFKA-957.v1.patch
>
>
> Currently, MirrorMaker only propagates the message to the target cluster, but 
> not the associated key.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Commented] (KAFKA-885) sbt package builds two kafka jars

2013-07-11 Thread Jun Rao (JIRA)

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

Jun Rao commented on KAFKA-885:
---

Thanks for the patch. Is there away to not generating the top level jar at all? 
If not, I guess this patch is also ok and we can tak it in 0.8.

> sbt package builds two kafka jars
> -
>
> Key: KAFKA-885
> URL: https://issues.apache.org/jira/browse/KAFKA-885
> Project: Kafka
>  Issue Type: Improvement
>  Components: packaging
>Affects Versions: 0.8
>Reporter: Jun Rao
>Priority: Blocker
>  Labels: patch
> Attachments: 885-v1.patch
>
>
> After recent changes to sbt, ./sbt package now builds two kafka jars. One 
> under
> ./target/scala-2.8.0/kafka_2.8.0-0.8.0-SNAPSHOT.jar
> and another under
> ./core/target/scala-2.8.0/kafka_2.8.0-0.8.0-SNAPSHOT.jar
> The former seems incorrect.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


Re: Running 0.8 against 0.7 data and zk

2013-07-11 Thread Jun Rao
No. If you point an 0.8 broker to a ZK with 0.7 layout, it will still hit
some weird exception, maybe not exactly the same as specified in the jira.

Thanks,

Jun


On Thu, Jul 11, 2013 at 3:15 PM, Jay Kreps  wrote:

> This JIRA has a patch but I thought we already did something like this:
> https://issues.apache.org/jira/browse/KAFKA-686
>
> Am I just confused?
>
> -Jay
>


[jira] [Commented] (KAFKA-885) sbt package builds two kafka jars

2013-07-11 Thread Ashwanth Fernando (JIRA)

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

Ashwanth Fernando commented on KAFKA-885:
-

Let me see if we can do that. Will get back to you.

> sbt package builds two kafka jars
> -
>
> Key: KAFKA-885
> URL: https://issues.apache.org/jira/browse/KAFKA-885
> Project: Kafka
>  Issue Type: Improvement
>  Components: packaging
>Affects Versions: 0.8
>Reporter: Jun Rao
>Priority: Blocker
>  Labels: patch
> Attachments: 885-v1.patch
>
>
> After recent changes to sbt, ./sbt package now builds two kafka jars. One 
> under
> ./target/scala-2.8.0/kafka_2.8.0-0.8.0-SNAPSHOT.jar
> and another under
> ./core/target/scala-2.8.0/kafka_2.8.0-0.8.0-SNAPSHOT.jar
> The former seems incorrect.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira


[jira] [Comment Edited] (KAFKA-885) sbt package builds two kafka jars

2013-07-11 Thread Ashwanth Fernando (JIRA)

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

Ashwanth Fernando edited comment on KAFKA-885 at 7/12/13 6:58 AM:
--

"Is there away to not generating the top level jar at all?" - Let me see if we 
can do that. Will get back to you.

  was (Author: ashwanthferna...@gmail.com):
Let me see if we can do that. Will get back to you.
  
> sbt package builds two kafka jars
> -
>
> Key: KAFKA-885
> URL: https://issues.apache.org/jira/browse/KAFKA-885
> Project: Kafka
>  Issue Type: Improvement
>  Components: packaging
>Affects Versions: 0.8
>Reporter: Jun Rao
>Priority: Blocker
>  Labels: patch
> Attachments: 885-v1.patch
>
>
> After recent changes to sbt, ./sbt package now builds two kafka jars. One 
> under
> ./target/scala-2.8.0/kafka_2.8.0-0.8.0-SNAPSHOT.jar
> and another under
> ./core/target/scala-2.8.0/kafka_2.8.0-0.8.0-SNAPSHOT.jar
> The former seems incorrect.

--
This message is automatically generated by JIRA.
If you think it was sent incorrectly, please contact your JIRA administrators
For more information on JIRA, see: http://www.atlassian.com/software/jira