[jira] [Reopened] (KAFKA-2633) Default logging from tools to Stderr

2015-10-13 Thread Ewen Cheslack-Postava (JIRA)

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

Ewen Cheslack-Postava reopened KAFKA-2633:
--

Commented on PR about potential compatibility issues. Would be glad to see this 
closed with no issue, but I think this patch potentially breaks scripts using 
using a variety of existing CLI tools. Would like to at least see some 
description of potential impact.

> Default logging from tools to Stderr
> 
>
> Key: KAFKA-2633
> URL: https://issues.apache.org/jira/browse/KAFKA-2633
> Project: Kafka
>  Issue Type: Bug
>Reporter: Grant Henke
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> Currently the default logging for tool is stdout, this can make parsing the 
> output  difficult in cases where exceptions are thrown, or even when expected 
> logging messages are output. The most affected tool is the console-consumer 
> but others will have this issue as well. 
> Changing the output to stderr by default allows the user to redirect the 
> output to a log file without effecting the tools output.
> Note: Users can change the logging settings by setting $KAFKA_LOG4J_OPTS to 
> use the configuration of their choice. This Jira is to change the default to 
> be more user friendly



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


Re: [DISCUSS] KIP-32 - Add CreateTime and LogAppendTime to Kafka message

2015-10-13 Thread Jay Kreps
Here's my basic take:
- I agree it would be nice to have a notion of time baked in if it were
done right
- All the proposals so far seem pretty complex--I think they might make
things worse rather than better overall
- I think adding 2x8 byte timestamps to the message is probably a
non-starter from a size perspective
- Even if it isn't in the message, having two notions of time that control
different things is a bit confusing
- The mechanics of basing retention etc on log append time when that's not
in the log seem complicated

To that end here is a possible 4th option. Let me know what you think.

The basic idea is that the message creation time is closest to what the
user actually cares about but is dangerous if set wrong. So rather than
substitute another notion of time, let's try to ensure the correctness of
message creation time by preventing arbitrarily bad message creation times.

First, let's see if we can agree that log append time is not something
anyone really cares about but rather an implementation detail. The
timestamp that matters to the user is when the message occurred (the
creation time). The log append time is basically just an approximation to
this on the assumption that the message creation and the message receive on
the server occur pretty close together and the reason to prefer .

But as these values diverge the issue starts to become apparent. Say you
set the retention to one week and then mirror data from a topic containing
two years of retention. Your intention is clearly to keep the last week,
but because the mirroring is appending right now you will keep two years.

The reason we are liking log append time is because we are (justifiably)
concerned that in certain situations the creation time may not be
trustworthy. This same problem exists on the servers but there are fewer
servers and they just run the kafka code so it is less of an issue.

There are two possible ways to handle this:

   1. Just tell people to add size based retention. I think this is not
   entirely unreasonable, we're basically saying we retain data based on the
   timestamp you give us in the data. If you give us bad data we will retain
   it for a bad amount of time. If you want to ensure we don't retain "too
   much" data, define "too much" by setting a time-based retention setting.
   This is not entirely unreasonable but kind of suffers from a "one bad
   apple" problem in a very large environment.
   2. Prevent bad timestamps. In general we can't say a timestamp is bad.
   However the definition we're implicitly using is that we think there are a
   set of topics/clusters where the creation timestamp should always be "very
   close" to the log append timestamp. This is true for data sources that have
   no buffering capability (which at LinkedIn is very common, but is more rare
   elsewhere). The solution in this case would be to allow a setting along the
   lines of max.append.delay which checks the creation timestamp against the
   server time to look for too large a divergence. The solution would either
   be to reject the message or to override it with the server time.

So in LI's environment you would configure the clusters used for direct,
unbuffered, message production (e.g. tracking and metrics local) to enforce
a reasonably aggressive timestamp bound (say 10 mins), and all other
clusters would just inherent these.

The downside of this approach is requiring the special configuration.
However I think in 99% of environments this could be skipped entirely, it's
only when the ratio of clients to servers gets so massive that you need to
do this. The primary upside is that you have a single authoritative notion
of time which is closest to what a user would want and is stored directly
in the message.

I'm also assuming there is a workable approach for indexing non-monotonic
timestamps, though I haven't actually worked that out.

-Jay

On Mon, Oct 5, 2015 at 8:52 PM, Jiangjie Qin 
wrote:

> Bumping up this thread although most of the discussion were on the
> discussion thread of KIP-31 :)
>
> I just updated the KIP page to add detailed solution for the option (Option
> 3) that does not expose the LogAppendTime to user.
>
>
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-32+-+Add+CreateTime+and+LogAppendTime+to+Kafka+message
>
> The option has a minor change to the fetch request to allow fetching time
> index entry as well. I kind of like this solution because its just doing
> what we need without introducing other things.
>
> It will be great to see what are the feedback. I can explain more during
> tomorrow's KIP hangout.
>
> Thanks,
>
> Jiangjie (Becket) Qin
>
> On Thu, Sep 10, 2015 at 2:47 PM, Jiangjie Qin  wrote:
>
> > Hi Jay,
> >
> > I just copy/pastes here your feedback on the timestamp proposal that was
> > in the discussion thread of KIP-31. Please see the replies inline.
> > The main change I made compared with previous proposal is to add both
> > CreateTime and LogAppendTime to the me

Re: [DISCUSS] KIP-32 - Add CreateTime and LogAppendTime to Kafka message

2015-10-13 Thread Jay Kreps
I think it should be possible to index out-of-order timestamps. The
timestamp index would be similar to the offset index, a memory mapped file
appended to as part of the log append, but would have the format
  timestamp offset
The timestamp entries would be monotonic and as with the offset index would
be no more often then every 4k (or some configurable threshold to keep the
index small--actually for timestamp it could probably be much more sparse
than 4k).

A search for a timestamp t yields an offset o before which no prior message
has a timestamp >= t. In other words if you read the log starting with o
you are guaranteed not to miss any messages occurring at t or later though
you may get many before t (due to out-of-orderness). Unlike the offset
index this bound doesn't really have to be tight (i.e. probably no need to
go search the log itself, though you could).

-Jay

On Tue, Oct 13, 2015 at 12:32 AM, Jay Kreps  wrote:

> Here's my basic take:
> - I agree it would be nice to have a notion of time baked in if it were
> done right
> - All the proposals so far seem pretty complex--I think they might make
> things worse rather than better overall
> - I think adding 2x8 byte timestamps to the message is probably a
> non-starter from a size perspective
> - Even if it isn't in the message, having two notions of time that control
> different things is a bit confusing
> - The mechanics of basing retention etc on log append time when that's not
> in the log seem complicated
>
> To that end here is a possible 4th option. Let me know what you think.
>
> The basic idea is that the message creation time is closest to what the
> user actually cares about but is dangerous if set wrong. So rather than
> substitute another notion of time, let's try to ensure the correctness of
> message creation time by preventing arbitrarily bad message creation times.
>
> First, let's see if we can agree that log append time is not something
> anyone really cares about but rather an implementation detail. The
> timestamp that matters to the user is when the message occurred (the
> creation time). The log append time is basically just an approximation to
> this on the assumption that the message creation and the message receive on
> the server occur pretty close together and the reason to prefer .
>
> But as these values diverge the issue starts to become apparent. Say you
> set the retention to one week and then mirror data from a topic containing
> two years of retention. Your intention is clearly to keep the last week,
> but because the mirroring is appending right now you will keep two years.
>
> The reason we are liking log append time is because we are (justifiably)
> concerned that in certain situations the creation time may not be
> trustworthy. This same problem exists on the servers but there are fewer
> servers and they just run the kafka code so it is less of an issue.
>
> There are two possible ways to handle this:
>
>1. Just tell people to add size based retention. I think this is not
>entirely unreasonable, we're basically saying we retain data based on the
>timestamp you give us in the data. If you give us bad data we will retain
>it for a bad amount of time. If you want to ensure we don't retain "too
>much" data, define "too much" by setting a time-based retention setting.
>This is not entirely unreasonable but kind of suffers from a "one bad
>apple" problem in a very large environment.
>2. Prevent bad timestamps. In general we can't say a timestamp is bad.
>However the definition we're implicitly using is that we think there are a
>set of topics/clusters where the creation timestamp should always be "very
>close" to the log append timestamp. This is true for data sources that have
>no buffering capability (which at LinkedIn is very common, but is more rare
>elsewhere). The solution in this case would be to allow a setting along the
>lines of max.append.delay which checks the creation timestamp against the
>server time to look for too large a divergence. The solution would either
>be to reject the message or to override it with the server time.
>
> So in LI's environment you would configure the clusters used for direct,
> unbuffered, message production (e.g. tracking and metrics local) to enforce
> a reasonably aggressive timestamp bound (say 10 mins), and all other
> clusters would just inherent these.
>
> The downside of this approach is requiring the special configuration.
> However I think in 99% of environments this could be skipped entirely, it's
> only when the ratio of clients to servers gets so massive that you need to
> do this. The primary upside is that you have a single authoritative notion
> of time which is closest to what a user would want and is stored directly
> in the message.
>
> I'm also assuming there is a workable approach for indexing non-monotonic
> timestamps, though I haven't actually worked that out.
>
> -Jay
>
> On Mon, Oct 5, 

[jira] [Created] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread Ben Stopford (JIRA)
Ben Stopford created KAFKA-2637:
---

 Summary: Cipher suite setting should be configurable for SSL
 Key: KAFKA-2637
 URL: https://issues.apache.org/jira/browse/KAFKA-2637
 Project: Kafka
  Issue Type: Improvement
Reporter: Ben Stopford


Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
but ignored meaning you can't change cipher via the regular SSL Config. This 
should be enabled.



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


[GitHub] kafka pull request: KAFKA-2637: Cipher suite setting should be con...

2015-10-13 Thread benstopford
GitHub user benstopford opened a pull request:

https://github.com/apache/kafka/pull/301

KAFKA-2637: Cipher suite setting should be configurable for SSL

Enables Cipher suite setting. Code was previously reviewed by @ijuma, 
@harshach. Moving to an independent PR.

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/benstopford/kafka cipher-switch

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/301.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #301


commit 1ad5d1b631651d51328bd939c3856843536002c6
Author: benstopford 
Date:   2015-10-13T09:13:54Z

KAFKA-2637: Cipher suite setting should be configurable for SSL




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2637:
---

GitHub user benstopford opened a pull request:

https://github.com/apache/kafka/pull/301

KAFKA-2637: Cipher suite setting should be configurable for SSL

Enables Cipher suite setting. Code was previously reviewed by @ijuma, 
@harshach. Moving to an independent PR.

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/benstopford/kafka cipher-switch

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/301.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #301


commit 1ad5d1b631651d51328bd939c3856843536002c6
Author: benstopford 
Date:   2015-10-13T09:13:54Z

KAFKA-2637: Cipher suite setting should be configurable for SSL




> Cipher suite setting should be configurable for SSL
> ---
>
> Key: KAFKA-2637
> URL: https://issues.apache.org/jira/browse/KAFKA-2637
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Ben Stopford
>
> Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
> but ignored meaning you can't change cipher via the regular SSL Config. This 
> should be enabled.



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


[jira] [Updated] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread Ben Stopford (JIRA)

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

Ben Stopford updated KAFKA-2637:

Status: Patch Available  (was: In Progress)

> Cipher suite setting should be configurable for SSL
> ---
>
> Key: KAFKA-2637
> URL: https://issues.apache.org/jira/browse/KAFKA-2637
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Ben Stopford
>Assignee: Ben Stopford
>
> Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
> but ignored meaning you can't change cipher via the regular SSL Config. This 
> should be enabled.



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


[jira] [Work started] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread Ben Stopford (JIRA)

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

Work on KAFKA-2637 started by Ben Stopford.
---
> Cipher suite setting should be configurable for SSL
> ---
>
> Key: KAFKA-2637
> URL: https://issues.apache.org/jira/browse/KAFKA-2637
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Ben Stopford
>Assignee: Ben Stopford
>
> Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
> but ignored meaning you can't change cipher via the regular SSL Config. This 
> should be enabled.



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


[jira] [Assigned] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread Ben Stopford (JIRA)

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

Ben Stopford reassigned KAFKA-2637:
---

Assignee: Ben Stopford

> Cipher suite setting should be configurable for SSL
> ---
>
> Key: KAFKA-2637
> URL: https://issues.apache.org/jira/browse/KAFKA-2637
> Project: Kafka
>  Issue Type: Improvement
>Reporter: Ben Stopford
>Assignee: Ben Stopford
>
> Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
> but ignored meaning you can't change cipher via the regular SSL Config. This 
> should be enabled.



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


[jira] [Updated] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread Ben Stopford (JIRA)

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

Ben Stopford updated KAFKA-2637:

Component/s: security

> Cipher suite setting should be configurable for SSL
> ---
>
> Key: KAFKA-2637
> URL: https://issues.apache.org/jira/browse/KAFKA-2637
> Project: Kafka
>  Issue Type: Improvement
>  Components: security
>Reporter: Ben Stopford
>Assignee: Ben Stopford
>
> Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
> but ignored meaning you can't change cipher via the regular SSL Config. This 
> should be enabled.



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


[jira] [Resolved] (KAFKA-2564) SSL: Received fatal alert: handshake_failure occurs sporadically

2015-10-13 Thread Ben Stopford (JIRA)

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

Ben Stopford resolved KAFKA-2564.
-
Resolution: Duplicate
  Assignee: Ben Stopford

Duplicate of https://issues.apache.org/jira/browse/KAFKA-2504 which is resolved

> SSL: Received fatal alert: handshake_failure occurs sporadically 
> -
>
> Key: KAFKA-2564
> URL: https://issues.apache.org/jira/browse/KAFKA-2564
> Project: Kafka
>  Issue Type: Bug
>  Components: security
>Reporter: Ben Stopford
>Assignee: Ben Stopford
> Fix For: 0.9.0.0
>
>
> We sporadically get this error when SSL is enabled. It might be better if we 
> retried this rather than failing immediately on the error. 
> [2015-09-21 17:22:09,446] WARN Error in I/O with connection to 
> ip-172-31-34-157.eu-west-1.compute.internal/172.31.34.157 
> (org.apache.kafka.common.network.Selector)
> javax.net.ssl.SSLException: Received fatal alert: handshake_failure
>   at sun.security.ssl.Alerts.getSSLException(Alerts.java:208)
>   at sun.security.ssl.SSLEngineImpl.fatal(SSLEngineImpl.java:1639)
>   at sun.security.ssl.SSLEngineImpl.fatal(SSLEngineImpl.java:1607)
>   at sun.security.ssl.SSLEngineImpl.recvAlert(SSLEngineImpl.java:1776)
>   at sun.security.ssl.SSLEngineImpl.readRecord(SSLEngineImpl.java:1068)
>   at sun.security.ssl.SSLEngineImpl.readNetRecord(SSLEngineImpl.java:890)
>   at sun.security.ssl.SSLEngineImpl.unwrap(SSLEngineImpl.java:764)
>   at javax.net.ssl.SSLEngine.unwrap(SSLEngine.java:624)
>   at 
> org.apache.kafka.common.network.SSLTransportLayer.handshakeUnwrap(SSLTransportLayer.java:377)
>   at 
> org.apache.kafka.common.network.SSLTransportLayer.handshake(SSLTransportLayer.java:247)
>   at 
> org.apache.kafka.common.network.KafkaChannel.prepare(KafkaChannel.java:69)
>   at org.apache.kafka.common.network.Selector.poll(Selector.java:293)
>   at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:250)
>   at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.clientPoll(ConsumerNetworkClient.java:274)
>   at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:182)
>   at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:172)
>   at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.awaitMetadataUpdate(ConsumerNetworkClient.java:125)
>   at 
> org.apache.kafka.clients.consumer.internals.Coordinator.ensureCoordinatorKnown(Coordinator.java:214)
>   at 
> org.apache.kafka.clients.consumer.KafkaConsumer.pollOnce(KafkaConsumer.java:804)
>   at 
> org.apache.kafka.clients.consumer.KafkaConsumer.poll(KafkaConsumer.java:776)
>   at 
> kafka.tools.ConsumerPerformance$.consume(ConsumerPerformance.scala:122)
>   at kafka.tools.ConsumerPerformance$.main(ConsumerPerformance.scala:66)
>   at kafka.tools.ConsumerPerformance.main(ConsumerPerformance.scala)



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


[jira] [Created] (KAFKA-2638) ConsumerPerformance (kafka.tools) should support properties via file

2015-10-13 Thread Ben Stopford (JIRA)
Ben Stopford created KAFKA-2638:
---

 Summary: ConsumerPerformance (kafka.tools) should support 
properties via file
 Key: KAFKA-2638
 URL: https://issues.apache.org/jira/browse/KAFKA-2638
 Project: Kafka
  Issue Type: Improvement
  Components: security
Reporter: Ben Stopford
Priority: Blocker


ConsumerPerformance should support properties via a properties file
So that freeform properties can be provided (for example SSL Config)

The properties provided in the file should be overridden by identical 
properties supplied on the command line. 

The file should be named consumer.config to match producer.config in 
ProducerPerformance



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


[GitHub] kafka pull request: KAFKA-2638: Added default properties file to C...

2015-10-13 Thread benstopford
GitHub user benstopford opened a pull request:

https://github.com/apache/kafka/pull/302

KAFKA-2638: Added default properties file to ConsumerPerformance

Blocker for SSL integration

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/benstopford/kafka KAFKA-2638

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/302.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #302


commit f44ea477455adf72985da47c0e48fbd392a00331
Author: benstopford 
Date:   2015-10-13T11:44:01Z

KAFKA-2638: Added default properties file via consumer.config command line 
option




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2638) ConsumerPerformance (kafka.tools) should support properties via file

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2638:
---

GitHub user benstopford opened a pull request:

https://github.com/apache/kafka/pull/302

KAFKA-2638: Added default properties file to ConsumerPerformance

Blocker for SSL integration

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/benstopford/kafka KAFKA-2638

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/302.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #302


commit f44ea477455adf72985da47c0e48fbd392a00331
Author: benstopford 
Date:   2015-10-13T11:44:01Z

KAFKA-2638: Added default properties file via consumer.config command line 
option




> ConsumerPerformance (kafka.tools) should support properties via file
> 
>
> Key: KAFKA-2638
> URL: https://issues.apache.org/jira/browse/KAFKA-2638
> Project: Kafka
>  Issue Type: Improvement
>  Components: security
>Reporter: Ben Stopford
>Priority: Blocker
>
> ConsumerPerformance should support properties via a properties file
> So that freeform properties can be provided (for example SSL Config)
> The properties provided in the file should be overridden by identical 
> properties supplied on the command line. 
> The file should be named consumer.config to match producer.config in 
> ProducerPerformance



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


[GitHub] kafka pull request: KAFKA-2638: Added default properties file to C...

2015-10-13 Thread benstopford
Github user benstopford closed the pull request at:

https://github.com/apache/kafka/pull/302


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2638) ConsumerPerformance (kafka.tools) should support properties via file

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2638:
---

Github user benstopford closed the pull request at:

https://github.com/apache/kafka/pull/302


> ConsumerPerformance (kafka.tools) should support properties via file
> 
>
> Key: KAFKA-2638
> URL: https://issues.apache.org/jira/browse/KAFKA-2638
> Project: Kafka
>  Issue Type: Improvement
>  Components: security
>Reporter: Ben Stopford
>Priority: Blocker
>
> ConsumerPerformance should support properties via a properties file
> So that freeform properties can be provided (for example SSL Config)
> The properties provided in the file should be overridden by identical 
> properties supplied on the command line. 
> The file should be named consumer.config to match producer.config in 
> ProducerPerformance



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


[jira] [Resolved] (KAFKA-2638) ConsumerPerformance (kafka.tools) should support properties via file

2015-10-13 Thread Ben Stopford (JIRA)

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

Ben Stopford resolved KAFKA-2638.
-
Resolution: Duplicate

> ConsumerPerformance (kafka.tools) should support properties via file
> 
>
> Key: KAFKA-2638
> URL: https://issues.apache.org/jira/browse/KAFKA-2638
> Project: Kafka
>  Issue Type: Improvement
>  Components: security
>Reporter: Ben Stopford
>Priority: Blocker
>
> ConsumerPerformance should support properties via a properties file
> So that freeform properties can be provided (for example SSL Config)
> The properties provided in the file should be overridden by identical 
> properties supplied on the command line. 
> The file should be named consumer.config to match producer.config in 
> ProducerPerformance



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


[jira] [Created] (KAFKA-2639) Refactoring of ZkUtils

2015-10-13 Thread Flavio Junqueira (JIRA)
Flavio Junqueira created KAFKA-2639:
---

 Summary: Refactoring of ZkUtils
 Key: KAFKA-2639
 URL: https://issues.apache.org/jira/browse/KAFKA-2639
 Project: Kafka
  Issue Type: Sub-task
Reporter: Flavio Junqueira
Assignee: Flavio Junqueira


Refactoring of ZkUtils to make the changes of KAFKA-1695 testable.



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


[jira] [Created] (KAFKA-2640) Add tests for ZK authentication

2015-10-13 Thread Flavio Junqueira (JIRA)
Flavio Junqueira created KAFKA-2640:
---

 Summary: Add tests for ZK authentication
 Key: KAFKA-2640
 URL: https://issues.apache.org/jira/browse/KAFKA-2640
 Project: Kafka
  Issue Type: Sub-task
Reporter: Flavio Junqueira
Assignee: Flavio Junqueira


Add tests for KAKA-2639.



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


[jira] [Created] (KAFKA-2641) Upgrade path for ZK authentication

2015-10-13 Thread Flavio Junqueira (JIRA)
Flavio Junqueira created KAFKA-2641:
---

 Summary: Upgrade path for ZK authentication
 Key: KAFKA-2641
 URL: https://issues.apache.org/jira/browse/KAFKA-2641
 Project: Kafka
  Issue Type: Sub-task
Reporter: Flavio Junqueira
Assignee: Flavio Junqueira


Add necessary configuration and scripting to make sure that existing clusters 
can turn security on with minimal disruption.



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


[GitHub] kafka pull request: Kafka 2639: Refactoring of ZkUtils

2015-10-13 Thread fpj
GitHub user fpj opened a pull request:

https://github.com/apache/kafka/pull/303

Kafka 2639: Refactoring of ZkUtils

I've split the work of KAFKA-1695 because this refactoring touches a large 
number of files. Most of the changes are trivial, but I feel it will be easier 
to review this way.

This pull request includes the one @Parth-Brahmbhatt started to address 
KAFKA-1695.  

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/fpj/kafka KAFKA-2639

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/303.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #303


commit 6a1ca42c41f0e577e33bf92cdc6aa6ec3a8da237
Author: flavio junqueira 
Date:   2015-10-12T21:55:10Z

Initial pass, main code compiles

commit afeafabdcefc2dd93f28ab5e23041be7ebe08f3b
Author: flavio junqueira 
Date:   2015-10-13T12:10:43Z

Changes to tests to accomodate the refactoring of ZkUtils.

commit 66b116aace0990182d76b6591b50491f072b95cb
Author: flavio junqueira 
Date:   2015-10-13T12:59:06Z

Removed whitespaces.




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-1695) Authenticate connection to Zookeeper

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-1695:
---

GitHub user fpj opened a pull request:

https://github.com/apache/kafka/pull/303

Kafka 2639: Refactoring of ZkUtils

I've split the work of KAFKA-1695 because this refactoring touches a large 
number of files. Most of the changes are trivial, but I feel it will be easier 
to review this way.

This pull request includes the one @Parth-Brahmbhatt started to address 
KAFKA-1695.  

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/fpj/kafka KAFKA-2639

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/303.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #303


commit 6a1ca42c41f0e577e33bf92cdc6aa6ec3a8da237
Author: flavio junqueira 
Date:   2015-10-12T21:55:10Z

Initial pass, main code compiles

commit afeafabdcefc2dd93f28ab5e23041be7ebe08f3b
Author: flavio junqueira 
Date:   2015-10-13T12:10:43Z

Changes to tests to accomodate the refactoring of ZkUtils.

commit 66b116aace0990182d76b6591b50491f072b95cb
Author: flavio junqueira 
Date:   2015-10-13T12:59:06Z

Removed whitespaces.




> Authenticate connection to Zookeeper
> 
>
> Key: KAFKA-1695
> URL: https://issues.apache.org/jira/browse/KAFKA-1695
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Reporter: Jay Kreps
>Assignee: Parth Brahmbhatt
> Fix For: 0.9.0.0
>
>
> We need to make it possible to secure the Zookeeper cluster Kafka is using. 
> This would make use of the normal authentication ZooKeeper provides. 
> ZooKeeper supports a variety of authentication mechanisms so we will need to 
> figure out what has to be passed in to the zookeeper client.
> The intention is that when the current round of client work is done it should 
> be possible to run without clients needing access to Zookeeper so all we need 
> here is to make it so that only the Kafka cluster is able to read and write 
> to the Kafka znodes  (we shouldn't need to set any kind of acl on a per-znode 
> basis).



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


[jira] [Updated] (KAFKA-2581) Run some existing ducktape tests with SSL-enabled clients and brokers

2015-10-13 Thread Rajini Sivaram (JIRA)

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

Rajini Sivaram updated KAFKA-2581:
--
Summary: Run some existing ducktape tests with SSL-enabled clients and 
brokers  (was: Run all existing ducktape tests with SSL-enabled clients and 
brokers)

> Run some existing ducktape tests with SSL-enabled clients and brokers
> -
>
> Key: KAFKA-2581
> URL: https://issues.apache.org/jira/browse/KAFKA-2581
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Affects Versions: 0.9.0.0
>Reporter: Rajini Sivaram
>Assignee: Rajini Sivaram
> Fix For: 0.9.0.0
>
>
> New ducktape tests for testing SSL are being added under KAFKA-2417. This 
> task will enable existing ducktape tests to be run with SSL-enabled brokers 
> and clients.



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


[jira] [Commented] (KAFKA-2581) Run some existing ducktape tests with SSL-enabled clients and brokers

2015-10-13 Thread Rajini Sivaram (JIRA)

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

Rajini Sivaram commented on KAFKA-2581:
---

Tests that have been SSL enabled:
# Console consumer sanity test: combinations of SSL for clients and interbroker 
communication
# Benchmarks : combinations of SSL for clients and interbroker communication
# Replication tests with SSL for interbroker communication

Will raise a JIRA for tests which need KAFKA-2603 to run with the new consumer
# Turn on SSL for clients as well in replication tests
# Run mirror maker tests with SSL



> Run some existing ducktape tests with SSL-enabled clients and brokers
> -
>
> Key: KAFKA-2581
> URL: https://issues.apache.org/jira/browse/KAFKA-2581
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Affects Versions: 0.9.0.0
>Reporter: Rajini Sivaram
>Assignee: Rajini Sivaram
> Fix For: 0.9.0.0
>
>
> New ducktape tests for testing SSL are being added under KAFKA-2417. This 
> task will enable existing ducktape tests to be run with SSL-enabled brokers 
> and clients.



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


[jira] [Commented] (KAFKA-2581) Run some existing ducktape tests with SSL-enabled clients and brokers

2015-10-13 Thread Ismael Juma (JIRA)

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

Ismael Juma commented on KAFKA-2581:


Thanks!

> Run some existing ducktape tests with SSL-enabled clients and brokers
> -
>
> Key: KAFKA-2581
> URL: https://issues.apache.org/jira/browse/KAFKA-2581
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Affects Versions: 0.9.0.0
>Reporter: Rajini Sivaram
>Assignee: Rajini Sivaram
> Fix For: 0.9.0.0
>
>
> New ducktape tests for testing SSL are being added under KAFKA-2417. This 
> task will enable existing ducktape tests to be run with SSL-enabled brokers 
> and clients.



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


[jira] [Commented] (KAFKA-2637) Cipher suite setting should be configurable for SSL

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2637:
---

Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/301


> Cipher suite setting should be configurable for SSL
> ---
>
> Key: KAFKA-2637
> URL: https://issues.apache.org/jira/browse/KAFKA-2637
> Project: Kafka
>  Issue Type: Improvement
>  Components: security
>Reporter: Ben Stopford
>Assignee: Ben Stopford
>
> Currently the Cipher Suite property (SSL_CIPHER_SUITES_CONFIG) is configured 
> but ignored meaning you can't change cipher via the regular SSL Config. This 
> should be enabled.



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


[GitHub] kafka pull request: KAFKA-2637: Cipher suite setting should be con...

2015-10-13 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/301


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Created] (KAFKA-2642) Run replication tests in ducktape with SSL for clients

2015-10-13 Thread Rajini Sivaram (JIRA)
Rajini Sivaram created KAFKA-2642:
-

 Summary: Run replication tests in ducktape with SSL for clients
 Key: KAFKA-2642
 URL: https://issues.apache.org/jira/browse/KAFKA-2642
 Project: Kafka
  Issue Type: Test
Reporter: Rajini Sivaram
Assignee: Rajini Sivaram
 Fix For: 0.9.0.0


Under KAFKA-2581, replication tests were parametrized to run with SSL for 
interbroker communication, but not for clients. When KAFKA-2603 is committed, 
the tests should be able to use SSL for clients as well,



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


[jira] [Created] (KAFKA-2643) Run mirror maker tests in ducktape with SSL

2015-10-13 Thread Rajini Sivaram (JIRA)
Rajini Sivaram created KAFKA-2643:
-

 Summary: Run mirror maker tests in ducktape with SSL
 Key: KAFKA-2643
 URL: https://issues.apache.org/jira/browse/KAFKA-2643
 Project: Kafka
  Issue Type: Test
Reporter: Rajini Sivaram
Assignee: Rajini Sivaram
 Fix For: 0.9.0.0


Mirror maker tests are currently run only with PLAINTEXT. Should be run with 
SSL as well. This requires console consumer timeout in new consumers which is 
being added in KAFKA-2603



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


[jira] [Created] (KAFKA-2644) Run relevant ducktape tests with SASL_PLAIN and SASL_SSL

2015-10-13 Thread Ismael Juma (JIRA)
Ismael Juma created KAFKA-2644:
--

 Summary: Run relevant ducktape tests with SASL_PLAIN and SASL_SSL
 Key: KAFKA-2644
 URL: https://issues.apache.org/jira/browse/KAFKA-2644
 Project: Kafka
  Issue Type: Sub-task
Reporter: Ismael Juma
Priority: Critical
 Fix For: 0.9.0.0


We need to define which of the existing ducktape tests are relevant. cc 
[~rsivaram]



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


[jira] [Commented] (KAFKA-2643) Run mirror maker tests in ducktape with SSL

2015-10-13 Thread Ismael Juma (JIRA)

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

Ismael Juma commented on KAFKA-2643:


This probably depends on KAFKA-2452 too, right?

> Run mirror maker tests in ducktape with SSL
> ---
>
> Key: KAFKA-2643
> URL: https://issues.apache.org/jira/browse/KAFKA-2643
> Project: Kafka
>  Issue Type: Test
>Reporter: Rajini Sivaram
>Assignee: Rajini Sivaram
> Fix For: 0.9.0.0
>
>
> Mirror maker tests are currently run only with PLAINTEXT. Should be run with 
> SSL as well. This requires console consumer timeout in new consumers which is 
> being added in KAFKA-2603



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


[jira] [Commented] (KAFKA-2643) Run mirror maker tests in ducktape with SSL

2015-10-13 Thread Rajini Sivaram (JIRA)

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

Rajini Sivaram commented on KAFKA-2643:
---

[~ijuma] Thank you, yes, this task depends on KAFKA-2452 as well.

> Run mirror maker tests in ducktape with SSL
> ---
>
> Key: KAFKA-2643
> URL: https://issues.apache.org/jira/browse/KAFKA-2643
> Project: Kafka
>  Issue Type: Test
>Reporter: Rajini Sivaram
>Assignee: Rajini Sivaram
> Fix For: 0.9.0.0
>
>
> Mirror maker tests are currently run only with PLAINTEXT. Should be run with 
> SSL as well. This requires console consumer timeout in new consumers which is 
> being added in KAFKA-2603



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


[jira] [Commented] (KAFKA-2644) Run relevant ducktape tests with SASL_PLAIN and SASL_SSL

2015-10-13 Thread Rajini Sivaram (JIRA)

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

Rajini Sivaram commented on KAFKA-2644:
---

[~ijuma] Are you planning to work on this? If not, I am happy to do this.

> Run relevant ducktape tests with SASL_PLAIN and SASL_SSL
> 
>
> Key: KAFKA-2644
> URL: https://issues.apache.org/jira/browse/KAFKA-2644
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Reporter: Ismael Juma
>Priority: Critical
> Fix For: 0.9.0.0
>
>
> We need to define which of the existing ducktape tests are relevant. cc 
> [~rsivaram]



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


[jira] [Commented] (KAFKA-2644) Run relevant ducktape tests with SASL_PLAIN and SASL_SSL

2015-10-13 Thread Ismael Juma (JIRA)

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

Ismael Juma commented on KAFKA-2644:


It would be awesome if you could do this.

> Run relevant ducktape tests with SASL_PLAIN and SASL_SSL
> 
>
> Key: KAFKA-2644
> URL: https://issues.apache.org/jira/browse/KAFKA-2644
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Reporter: Ismael Juma
>Priority: Critical
> Fix For: 0.9.0.0
>
>
> We need to define which of the existing ducktape tests are relevant. cc 
> [~rsivaram]



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


Jenkins build is back to normal : kafka-trunk-jdk7 #681

2015-10-13 Thread Apache Jenkins Server
See 



[jira] [Commented] (KAFKA-2633) Default logging from tools to Stderr

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira commented on KAFKA-2633:
-

[~ewencp], thanks for highlighting a possible issue.

I'm not clear if the goal of the reopen is to discuss whether the change is a 
good idea (which [~granthenke] followed up on in the PR), or to request 
documentation for the change (which can be done in a separate JIRA).

> Default logging from tools to Stderr
> 
>
> Key: KAFKA-2633
> URL: https://issues.apache.org/jira/browse/KAFKA-2633
> Project: Kafka
>  Issue Type: Bug
>Reporter: Grant Henke
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> Currently the default logging for tool is stdout, this can make parsing the 
> output  difficult in cases where exceptions are thrown, or even when expected 
> logging messages are output. The most affected tool is the console-consumer 
> but others will have this issue as well. 
> Changing the output to stderr by default allows the user to redirect the 
> output to a log file without effecting the tools output.
> Note: Users can change the logging settings by setting $KAFKA_LOG4J_OPTS to 
> use the configuration of their choice. This Jira is to change the default to 
> be more user friendly



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


[jira] [Commented] (KAFKA-2633) Default logging from tools to Stderr

2015-10-13 Thread Brock Noland (JIRA)

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

Brock Noland commented on KAFKA-2633:
-

>From a user perspective, I'd like to see this in ASAP. I almost blew a gasket 
>when I first realized Kafka was logging to stdout.

> Default logging from tools to Stderr
> 
>
> Key: KAFKA-2633
> URL: https://issues.apache.org/jira/browse/KAFKA-2633
> Project: Kafka
>  Issue Type: Bug
>Reporter: Grant Henke
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> Currently the default logging for tool is stdout, this can make parsing the 
> output  difficult in cases where exceptions are thrown, or even when expected 
> logging messages are output. The most affected tool is the console-consumer 
> but others will have this issue as well. 
> Changing the output to stderr by default allows the user to redirect the 
> output to a log file without effecting the tools output.
> Note: Users can change the logging settings by setting $KAFKA_LOG4J_OPTS to 
> use the configuration of their choice. This Jira is to change the default to 
> be more user friendly



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


[jira] [Commented] (KAFKA-2633) Default logging from tools to Stderr

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke commented on KAFKA-2633:


_Duplicating my comment here_

I am happy to send a docs patch listing a potentially breaking change. It can 
be a new jira or a follow up patch. We should likely enumerate the breaking 
changes I listed in my PR comment somewhere as well.

Obligatory XKCD comic: https://xkcd.com/1172/

> Default logging from tools to Stderr
> 
>
> Key: KAFKA-2633
> URL: https://issues.apache.org/jira/browse/KAFKA-2633
> Project: Kafka
>  Issue Type: Bug
>Reporter: Grant Henke
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> Currently the default logging for tool is stdout, this can make parsing the 
> output  difficult in cases where exceptions are thrown, or even when expected 
> logging messages are output. The most affected tool is the console-consumer 
> but others will have this issue as well. 
> Changing the output to stderr by default allows the user to redirect the 
> output to a log file without effecting the tools output.
> Note: Users can change the logging settings by setting $KAFKA_LOG4J_OPTS to 
> use the configuration of their choice. This Jira is to change the default to 
> be more user friendly



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


[jira] [Commented] (KAFKA-2633) Default logging from tools to Stderr

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira commented on KAFKA-2633:
-

[~brocknoland] - technically it was fixed, the change was merged yesterday and 
was not rolled back. It will be in the next release.



> Default logging from tools to Stderr
> 
>
> Key: KAFKA-2633
> URL: https://issues.apache.org/jira/browse/KAFKA-2633
> Project: Kafka
>  Issue Type: Bug
>Reporter: Grant Henke
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> Currently the default logging for tool is stdout, this can make parsing the 
> output  difficult in cases where exceptions are thrown, or even when expected 
> logging messages are output. The most affected tool is the console-consumer 
> but others will have this issue as well. 
> Changing the output to stderr by default allows the user to redirect the 
> output to a log file without effecting the tools output.
> Note: Users can change the logging settings by setting $KAFKA_LOG4J_OPTS to 
> use the configuration of their choice. This Jira is to change the default to 
> be more user friendly



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


[jira] [Created] (KAFKA-2645) Document potentially breaking changes in the release notes for 0.9.0

2015-10-13 Thread Grant Henke (JIRA)
Grant Henke created KAFKA-2645:
--

 Summary: Document potentially breaking changes in the release 
notes for 0.9.0
 Key: KAFKA-2645
 URL: https://issues.apache.org/jira/browse/KAFKA-2645
 Project: Kafka
  Issue Type: Improvement
Reporter: Grant Henke
Assignee: Grant Henke
 Fix For: 0.9.0.0






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


[jira] [Resolved] (KAFKA-2633) Default logging from tools to Stderr

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke resolved KAFKA-2633.

Resolution: Fixed

> Default logging from tools to Stderr
> 
>
> Key: KAFKA-2633
> URL: https://issues.apache.org/jira/browse/KAFKA-2633
> Project: Kafka
>  Issue Type: Bug
>Reporter: Grant Henke
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> Currently the default logging for tool is stdout, this can make parsing the 
> output  difficult in cases where exceptions are thrown, or even when expected 
> logging messages are output. The most affected tool is the console-consumer 
> but others will have this issue as well. 
> Changing the output to stderr by default allows the user to redirect the 
> output to a log file without effecting the tools output.
> Note: Users can change the logging settings by setting $KAFKA_LOG4J_OPTS to 
> use the configuration of their choice. This Jira is to change the default to 
> be more user friendly



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


[jira] [Created] (KAFKA-2646) Re-enable altering topic config in the topics command to maintain backwards compatibility

2015-10-13 Thread Grant Henke (JIRA)
Grant Henke created KAFKA-2646:
--

 Summary: Re-enable altering topic config in the topics command to 
maintain backwards compatibility
 Key: KAFKA-2646
 URL: https://issues.apache.org/jira/browse/KAFKA-2646
 Project: Kafka
  Issue Type: Bug
  Components: tools
Affects Versions: 0.9.0.0
Reporter: Grant Henke
Assignee: Grant Henke
Priority: Critical
 Fix For: 0.9.0.0


KAFKA-2205 removed the ability to alter topic configs from the topic command. 
This jira is to add the functionality back, logging that it is deprecated for 
future release changes. 

This prevents scripts from breaking in 0.9.0 release. 

If you have any thoughts on how this should be handled please discuss in the 
comments below. 



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


[jira] [Assigned] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke reassigned KAFKA-2536:
--

Assignee: Grant Henke

> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>Reporter: Gwen Shapira
>Assignee: Grant Henke
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


[jira] [Resolved] (KAFKA-2646) Re-enable altering topic config in the topics command to maintain backwards compatibility

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke resolved KAFKA-2646.

Resolution: Duplicate

> Re-enable altering topic config in the topics command to maintain backwards 
> compatibility
> -
>
> Key: KAFKA-2646
> URL: https://issues.apache.org/jira/browse/KAFKA-2646
> Project: Kafka
>  Issue Type: Bug
>  Components: tools
>Affects Versions: 0.9.0.0
>Reporter: Grant Henke
>Assignee: Grant Henke
>Priority: Critical
> Fix For: 0.9.0.0
>
>
> KAFKA-2205 removed the ability to alter topic configs from the topic command. 
> This jira is to add the functionality back, logging that it is deprecated for 
> future release changes. 
> This prevents scripts from breaking in 0.9.0 release. 
> If you have any thoughts on how this should be handled please discuss in the 
> comments below. 



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


[GitHub] kafka pull request: KAFKA-2372: Add Kafka-backed storage of Copyca...

2015-10-13 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/241


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Updated] (KAFKA-2372) Copycat distributed config storage

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira updated KAFKA-2372:

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

Issue resolved by pull request 241
[https://github.com/apache/kafka/pull/241]

> Copycat distributed config storage
> --
>
> Key: KAFKA-2372
> URL: https://issues.apache.org/jira/browse/KAFKA-2372
> Project: Kafka
>  Issue Type: Sub-task
>  Components: copycat
>Reporter: Ewen Cheslack-Postava
>Assignee: Ewen Cheslack-Postava
> Fix For: 0.9.0.0
>
>
> Add a config storage mechanism to Copycat that works in distributed mode. 
> Copycat workers that start in distributed mode should use this implementation 
> by default.



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


[jira] [Commented] (KAFKA-2372) Copycat distributed config storage

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2372:
---

Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/241


> Copycat distributed config storage
> --
>
> Key: KAFKA-2372
> URL: https://issues.apache.org/jira/browse/KAFKA-2372
> Project: Kafka
>  Issue Type: Sub-task
>  Components: copycat
>Reporter: Ewen Cheslack-Postava
>Assignee: Ewen Cheslack-Postava
> Fix For: 0.9.0.0
>
>
> Add a config storage mechanism to Copycat that works in distributed mode. 
> Copycat workers that start in distributed mode should use this implementation 
> by default.



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


[jira] [Created] (KAFKA-2647) Migrate System Tools to work with SSL

2015-10-13 Thread Ben Stopford (JIRA)
Ben Stopford created KAFKA-2647:
---

 Summary: Migrate System Tools to work with SSL
 Key: KAFKA-2647
 URL: https://issues.apache.org/jira/browse/KAFKA-2647
 Project: Kafka
  Issue Type: Improvement
  Components: security
Reporter: Ben Stopford


The following system tools won't work with SSL enabled brokers. They should 
either be directly ported over or we should provide some equivalent 
functionality. 

- ReplicaVerificationTool (SimpleConsumer + Producer (via ClientUtils))
- ReplayLogProducer (HL Consumer / New Producer)
- ConsumerOffsetChecker (Broker Channel)
- GetOffsetShell (SimpleConsumer, Old Producer used for TopicMetadata fetch)
- SimpleConsumerShell (SimpleConsumer)



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


[jira] [Updated] (KAFKA-2620) Introduce Scalariform

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke updated KAFKA-2620:
---
Status: Patch Available  (was: Open)

> Introduce Scalariform
> -
>
> Key: KAFKA-2620
> URL: https://issues.apache.org/jira/browse/KAFKA-2620
> Project: Kafka
>  Issue Type: Bug
>  Components: build
>Reporter: Grant Henke
>Assignee: Grant Henke
>
> Many of our reviews include nit comments related to Scala style. Adding 
> [Scalariform|https://github.com/daniel-trinh/scalariform] allows us to 
> reformat the code based on configurable standards at build time, ensuring 
> uniform readability and a short review/commit cycle. 
> I expect this will have some discussion around the rules we would like to 
> include, and if we actually want to adopt this. I will submit a sample patch 
> to start the discussion.



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


[jira] [Commented] (KAFKA-2620) Introduce Scalariform

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira commented on KAFKA-2620:
-

Is there any benefit for getting this in before we agree on which style 
standard to configure?

> Introduce Scalariform
> -
>
> Key: KAFKA-2620
> URL: https://issues.apache.org/jira/browse/KAFKA-2620
> Project: Kafka
>  Issue Type: Bug
>  Components: build
>Reporter: Grant Henke
>Assignee: Grant Henke
>
> Many of our reviews include nit comments related to Scala style. Adding 
> [Scalariform|https://github.com/daniel-trinh/scalariform] allows us to 
> reformat the code based on configurable standards at build time, ensuring 
> uniform readability and a short review/commit cycle. 
> I expect this will have some discussion around the rules we would like to 
> include, and if we actually want to adopt this. I will submit a sample patch 
> to start the discussion.



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


[jira] [Commented] (KAFKA-2636) Producer connectivity obscured connection failure logging

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira commented on KAFKA-2636:
-

This specific error is in debug level because the KafkaProducer will try 
connecting to multiple nodes in its broker-list and retry multiple times before 
giving up. We don't want to clutter the log with all the errors on every 
network hitch if the produce request was ultimately successful.

Does that make sense? 

> Producer connectivity obscured connection failure logging
> -
>
> Key: KAFKA-2636
> URL: https://issues.apache.org/jira/browse/KAFKA-2636
> Project: Kafka
>  Issue Type: Bug
>Affects Versions: 0.8.2.1, 0.8.2.2
> Environment: Windows 8 running java implementation of Kafka Producer
>Reporter: Jason Kania
>
> The Kafka Producer does not generate a visible exception when a connection 
> cannot be made. Instead DEBUG settings are required to observe the problem as 
> shown below:
> [2015-10-12 21:23:20,335] DEBUG Error connecting to node 0 at 
> 482f4769eed1:9092: (org.apache.kafka.clients.NetworkClient)
> java.io.IOException: Can't resolve address: 482f4769eed1:9092
>   at org.apache.kafka.common.network.Selector.connect(Selector.java:138)
>   at 
> org.apache.kafka.clients.NetworkClient.initiateConnect(NetworkClient.java:417)
>   at org.apache.kafka.clients.NetworkClient.ready(NetworkClient.java:116)
>   at 
> org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:165)
>   at 
> org.apache.kafka.clients.producer.internals.Sender.run(Sender.java:122)
>   at java.lang.Thread.run(Unknown Source)
> Caused by: java.nio.channels.UnresolvedAddressException
>   at sun.nio.ch.Net.checkAddress(Unknown Source)
>   at sun.nio.ch.SocketChannelImpl.connect(Unknown Source)
>   at org.apache.kafka.common.network.Selector.connect(Selector.java:135)
>   ... 5 more
> [2015-10-12 21:23:20,358] DEBUG Initiating connection to node 0 at 
> 482f4769eed1:9092. (org.apache.kafka.clients.NetworkClient)
> Secondly, the errors do not identify the node by IP address making error 
> investigation more difficult especially when learning to use Kafka.



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


[jira] [Assigned] (KAFKA-2647) Migrate System Tools to work with SSL

2015-10-13 Thread Ben Stopford (JIRA)

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

Ben Stopford reassigned KAFKA-2647:
---

Assignee: Ben Stopford

> Migrate System Tools to work with SSL
> -
>
> Key: KAFKA-2647
> URL: https://issues.apache.org/jira/browse/KAFKA-2647
> Project: Kafka
>  Issue Type: Improvement
>  Components: security
>Reporter: Ben Stopford
>Assignee: Ben Stopford
>
> The following system tools won't work with SSL enabled brokers. They should 
> either be directly ported over or we should provide some equivalent 
> functionality. 
> - ReplicaVerificationTool (SimpleConsumer + Producer (via ClientUtils))
> - ReplayLogProducer (HL Consumer / New Producer)
> - ConsumerOffsetChecker (Broker Channel)
> - GetOffsetShell (SimpleConsumer, Old Producer used for TopicMetadata fetch)
> - SimpleConsumerShell (SimpleConsumer)



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


KIP-28 does not allow Processor to specify partition of output message

2015-10-13 Thread Randall Hauch
The new streams API added with KIP-28 is great. I’ve been using it on a
prototype for a few weeks, and I’m looking forward to it being included in
0.9.0. However, at the moment, a Processor implementation is not able to
specify the partition number when it outputs messages.

I’d be happy to log a JIRA and create a PR to add it to the API, but
without knowing all of the history I’m wondering if leaving it out of the
API was intentional.

Thoughts?

Best regards,

Randall Hauch


[jira] [Commented] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread Aditya Auradkar (JIRA)

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

Aditya Auradkar commented on KAFKA-2536:


[~gwenshap] - Thanks for reporting. Do we plan to keep this functionality in 
kafka-topic for future releases? In the future it will have to change to use 
the AlterConfig command to the brokers.

> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>Reporter: Gwen Shapira
>Assignee: Grant Henke
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


[jira] [Commented] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke commented on KAFKA-2536:


If we are deprecating it from the topic command, I can log that it is 
deprecated when used and we can remove it in the next release.

> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>Reporter: Gwen Shapira
>Assignee: Grant Henke
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


[jira] [Commented] (KAFKA-2620) Introduce Scalariform

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke commented on KAFKA-2620:


They should probably go in at a similar time, or at least the style should be 
agreed upon.

> Introduce Scalariform
> -
>
> Key: KAFKA-2620
> URL: https://issues.apache.org/jira/browse/KAFKA-2620
> Project: Kafka
>  Issue Type: Bug
>  Components: build
>Reporter: Grant Henke
>Assignee: Grant Henke
>
> Many of our reviews include nit comments related to Scala style. Adding 
> [Scalariform|https://github.com/daniel-trinh/scalariform] allows us to 
> reformat the code based on configurable standards at build time, ensuring 
> uniform readability and a short review/commit cycle. 
> I expect this will have some discussion around the rules we would like to 
> include, and if we actually want to adopt this. I will submit a sample patch 
> to start the discussion.



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


Build failed in Jenkins: kafka-trunk-jdk8 #24

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[cshapi] KAFKA-2372: Add Kafka-backed storage of Copycat configs.

--
[...truncated 362 lines...]
:kafka-trunk-jdk8:core:compileScala UP-TO-DATE
:kafka-trunk-jdk8:core:processResources UP-TO-DATE
:kafka-trunk-jdk8:core:classes UP-TO-DATE
:kafka-trunk-jdk8:log4j-appender:javadoc
:kafka-trunk-jdk8:core:javadoc
cache taskArtifacts.bin 
(/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/.gradle/2.4/taskArtifacts/taskArtifacts.bin)
 is corrupt. Discarding.
:kafka-trunk-jdk8:core:javadocJar
:kafka-trunk-jdk8:core:scaladoc
[ant:scaladoc] Element 
'/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/build/resources/main'
 does not exist.
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/server/KafkaServer.scala:277:
 warning: a pure expression does nothing in statement position; you may be 
omitting necessary parentheses
[ant:scaladoc] ControllerStats.uncleanLeaderElectionRate
[ant:scaladoc] ^
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/server/KafkaServer.scala:278:
 warning: a pure expression does nothing in statement position; you may be 
omitting necessary parentheses
[ant:scaladoc] ControllerStats.leaderElectionTimer
[ant:scaladoc] ^
[ant:scaladoc] warning: there were 14 feature warning(s); re-run with -feature 
for details
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/utils/ByteBoundedBlockingQueue.scala:72:
 warning: Could not find any member to link for 
"java.util.concurrent.BlockingQueue#offer".
[ant:scaladoc]   /**
[ant:scaladoc]   ^
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/utils/ByteBoundedBlockingQueue.scala:32:
 warning: Could not find any member to link for 
"java.util.concurrent.BlockingQueue#offer".
[ant:scaladoc]   /**
[ant:scaladoc]   ^
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/utils/ByteBoundedBlockingQueue.scala:137:
 warning: Could not find any member to link for 
"java.util.concurrent.BlockingQueue#poll".
[ant:scaladoc]   /**
[ant:scaladoc]   ^
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/utils/ByteBoundedBlockingQueue.scala:120:
 warning: Could not find any member to link for 
"java.util.concurrent.BlockingQueue#poll".
[ant:scaladoc]   /**
[ant:scaladoc]   ^
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/utils/ByteBoundedBlockingQueue.scala:97:
 warning: Could not find any member to link for 
"java.util.concurrent.BlockingQueue#put".
[ant:scaladoc]   /**
[ant:scaladoc]   ^
[ant:scaladoc] 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/utils/ByteBoundedBlockingQueue.scala:152:
 warning: Could not find any member to link for 
"java.util.concurrent.BlockingQueue#take".
[ant:scaladoc]   /**
[ant:scaladoc]   ^
[ant:scaladoc] 9 warnings found
:kafka-trunk-jdk8:core:scaladocJar
:kafka-trunk-jdk8:core:docsJar
:docsJar_2_11_7
Building project 'core' with Scala version 2.11.7
:kafka-trunk-jdk8:clients:compileJavawarning: [options] bootstrap class path 
not set in conjunction with -source 1.7
Note: 
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/clients/src/main/java/org/apache/kafka/clients/producer/KafkaProducer.java
 uses or overrides a deprecated API.
Note: Recompile with -Xlint:deprecation for details.
Note: Some input files use unchecked or unsafe operations.
Note: Recompile with -Xlint:unchecked for details.
1 warning

:kafka-trunk-jdk8:clients:processResources UP-TO-DATE
:kafka-trunk-jdk8:clients:classes
:kafka-trunk-jdk8:clients:determineCommitId UP-TO-DATE
:kafka-trunk-jdk8:clients:createVersionFile
:kafka-trunk-jdk8:clients:jar
:kafka-trunk-jdk8:clients:javadoc
:kafka-trunk-jdk8:log4j-appender:compileJavawarning: [options] bootstrap class 
path not set in conjunction with -source 1.7
1 warning

:kafka-trunk-jdk8:log4j-appender:processResources UP-TO-DATE
:kafka-trunk-jdk8:log4j-appender:classes
:kafka-trunk-jdk8:log4j-appender:jar
:kafka-trunk-jdk8:core:compileJava UP-TO-DATE
:kafka-trunk-jdk8:core:compileScalaJava HotSpot(TM) 64-Bit Server VM warning: 
ignoring option MaxPermSize=512m; support was removed in 8.0

/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/api/OffsetCommitRequest.scala:78:
 value DEFAULT_TIMESTAMP in object OffsetCommitRequest is deprecated: see 
corresponding Javadoc for more information.

org.apache.kafka.common.requests.OffsetCommitRequest.DEFAULT_TIMESTAMP
 ^
/x1/jenkins/jenkins-slave/workspace/kafka-trunk-jdk8/core/src/main/scala/kafka/common/OffsetMetadataAndError.scala:36:
 value DEFAULT_TIMESTAMP in object O

[jira] [Updated] (KAFKA-1804) Kafka network thread lacks top exception handler

2015-10-13 Thread Ismael Juma (JIRA)

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

Ismael Juma updated KAFKA-1804:
---
Fix Version/s: 0.9.0.0

> Kafka network thread lacks top exception handler
> 
>
> Key: KAFKA-1804
> URL: https://issues.apache.org/jira/browse/KAFKA-1804
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8.2.0
>Reporter: Oleg Golovin
>Priority: Critical
> Fix For: 0.9.0.0
>
>
> We have faced the problem that some kafka network threads may fail, so that 
> jstack attached to Kafka process showed fewer threads than we had defined in 
> our Kafka configuration. This leads to API requests processed by this thread 
> getting stuck unresponed.
> There were no error messages in the log regarding thread failure.
> We have examined Kafka code to find out there is no top try-catch block in 
> the network thread code, which could at least log possible errors.
> Could you add top-level try-catch block for the network thread, which should 
> recover network thread in case of exception?



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


[jira] [Resolved] (KAFKA-1804) Kafka network thread lacks top exception handler

2015-10-13 Thread Ismael Juma (JIRA)

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

Ismael Juma resolved KAFKA-1804.

Resolution: Fixed

Resolving this as it has been fixed.

> Kafka network thread lacks top exception handler
> 
>
> Key: KAFKA-1804
> URL: https://issues.apache.org/jira/browse/KAFKA-1804
> Project: Kafka
>  Issue Type: Bug
>  Components: core
>Affects Versions: 0.8.2.0
>Reporter: Oleg Golovin
>Priority: Critical
> Fix For: 0.9.0.0
>
>
> We have faced the problem that some kafka network threads may fail, so that 
> jstack attached to Kafka process showed fewer threads than we had defined in 
> our Kafka configuration. This leads to API requests processed by this thread 
> getting stuck unresponed.
> There were no error messages in the log regarding thread failure.
> We have examined Kafka code to find out there is no top try-catch block in 
> the network thread code, which could at least log possible errors.
> Could you add top-level try-catch block for the network thread, which should 
> recover network thread in case of exception?



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


Re: Kafka KIP meeting Oct 13 at 11:00am PST

2015-10-13 Thread Jun Rao
The following are the notes from today's meeting.

* 0.9.0 release: We discussed if KAFKA-2397 should be a blocker in 0.9.0.
Jason and Guozhang will follow up on the jira.
* KIP-32 and KIP-33: We discussed Jay's alternative proposal of just
keeping CreateTime in the message and having a config to control how far
off the CreateTime can be from the broker time. We will think a bit more on
this and Jiangjie will update the KIP wiki.
* KIP-36: We discussed an alternative approach of introducing a new broker
property to designate the rack. It's simpler and potentially can work in
the case when the broker to rack mapping is maintaining externally. We need
to make sure that we have an upgrade plan for this change. Allen will
update the KIP wiki.

The video will be uploaded soon in
https://cwiki.apache.org/confluence/display/KAFKA/Kafka
+Improvement+Proposals .

Thanks,

Jun


On Mon, Oct 12, 2015 at 1:41 PM, Jun Rao  wrote:

> Hi, Everyone,
>
> We will have a Kafka KIP meeting tomorrow at 11:00am PST. If you plan to
> attend but haven't received an invite, please let me know. The following is
> the agenda.
>
> Agenda:
> 1. Kafka 0.9.0 release
>
> and if time permits,
> 2. KIP-32: Add CreateTime and LogAppendTime to Kafka message
> 3. KIP-33: Add a time based log index
> 4. KIP-36: Add rack-aware support
>
> Thanks,
>
> Jun
>


[jira] [Commented] (KAFKA-2620) Introduce Scalariform

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira commented on KAFKA-2620:
-

So the plan is to wait until after 0.9.0 is released and then start a style 
discussion?

> Introduce Scalariform
> -
>
> Key: KAFKA-2620
> URL: https://issues.apache.org/jira/browse/KAFKA-2620
> Project: Kafka
>  Issue Type: Bug
>  Components: build
>Reporter: Grant Henke
>Assignee: Grant Henke
>
> Many of our reviews include nit comments related to Scala style. Adding 
> [Scalariform|https://github.com/daniel-trinh/scalariform] allows us to 
> reformat the code based on configurable standards at build time, ensuring 
> uniform readability and a short review/commit cycle. 
> I expect this will have some discussion around the rules we would like to 
> include, and if we actually want to adopt this. I will submit a sample patch 
> to start the discussion.



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


[jira] [Commented] (KAFKA-658) Implement "Exact Mirroring" functionality in mirror maker

2015-10-13 Thread Anil Sadineni (JIRA)

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

Anil Sadineni commented on KAFKA-658:
-

+1, I too have exact same use case

> Implement "Exact Mirroring" functionality in mirror maker
> -
>
> Key: KAFKA-658
> URL: https://issues.apache.org/jira/browse/KAFKA-658
> Project: Kafka
>  Issue Type: New Feature
>Reporter: Jay Kreps
>  Labels: project
>
> There are two ways to implement "mirroring" (i.e. replicating a topic from 
> one cluster to another):
> 1. Do a simple read from the source and write to the destination with no 
> attempt to maintain the same partitioning or offsets in the destination 
> cluster. In this case the destination cluster may have a different number of 
> partitions, and you can even read from many clusters to create a merged 
> cluster. This flexibility is nice. The downside is that since the 
> partitioning and offsets are not the same a consumer of the source cluster 
> has no equivalent position in the destination cluster. This is the style of 
> mirroring we have implemented in the mirror-maker tool and use for datacenter 
> replication today.
> 2. The second style of replication only would allow creating an exact replica 
> of a source cluster (i.e. all partitions and offsets exactly the same). The 
> nice thing about this is that the offsets and partitions would match exactly. 
> The downside is that it is not possible to merge multiple source clusters 
> this way or have different partitioning. We do not currently support this in 
> mirror maker.
> It would be nice to implement the second style as an option in mirror maker 
> as having an exact replica would be a nice option to have in the case where 
> you are replicating a single cluster only.
> There are some nuances: In order to maintain the exact offsets it is 
> important to guarantee that the producer never resends a message or loses a 
> message. As a result it would be important to have only a single producer for 
> each destination partition, and check the last produced message on startup 
> (using the getOffsets api) so that in the case of a hard crash messages that 
> are re-consumed are not re-emitted.



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


[jira] [Resolved] (KAFKA-2527) System Test for Quotas in Ducktape

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira resolved KAFKA-2527.
-
Resolution: Fixed

Issue resolved by pull request 275
[https://github.com/apache/kafka/pull/275]

> System Test for Quotas in Ducktape
> --
>
> Key: KAFKA-2527
> URL: https://issues.apache.org/jira/browse/KAFKA-2527
> Project: Kafka
>  Issue Type: Sub-task
>Reporter: Dong Lin
>Assignee: Dong Lin
>  Labels: quota
> Fix For: 0.9.0.0
>
>




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


[GitHub] kafka pull request: KAFKA-2527; System Test for Quotas in Ducktape

2015-10-13 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/275


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2527) System Test for Quotas in Ducktape

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2527:
---

Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/275


> System Test for Quotas in Ducktape
> --
>
> Key: KAFKA-2527
> URL: https://issues.apache.org/jira/browse/KAFKA-2527
> Project: Kafka
>  Issue Type: Sub-task
>Reporter: Dong Lin
>Assignee: Dong Lin
>  Labels: quota
> Fix For: 0.9.0.0
>
>




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


[jira] [Commented] (KAFKA-2397) leave group request

2015-10-13 Thread Onur Karaman (JIRA)

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

Onur Karaman commented on KAFKA-2397:
-

My pull request had diverged again from trunk, so I force pushed a rebase that 
just cleans up the conflicts.

> leave group request
> ---
>
> Key: KAFKA-2397
> URL: https://issues.apache.org/jira/browse/KAFKA-2397
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Reporter: Onur Karaman
>Assignee: Onur Karaman
>Priority: Minor
> Fix For: 0.9.0.0
>
>
> Let's say every consumer in a group has session timeout s. Currently, if a 
> consumer leaves the group, the worst case time to stabilize the group is 2s 
> (s to detect the consumer failure + s for the rebalance window). If a 
> consumer instead can declare they are leaving the group, the worst case time 
> to stabilize the group would just be the s associated with the rebalance 
> window.
> This is a low priority optimization!



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


[jira] [Commented] (KAFKA-658) Implement "Exact Mirroring" functionality in mirror maker

2015-10-13 Thread James Cheng (JIRA)

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

James Cheng commented on KAFKA-658:
---

I would also like this, so that consumers can transition from one cluster to 
another and be able to resume without missing or duplicating any records.


> Implement "Exact Mirroring" functionality in mirror maker
> -
>
> Key: KAFKA-658
> URL: https://issues.apache.org/jira/browse/KAFKA-658
> Project: Kafka
>  Issue Type: New Feature
>Reporter: Jay Kreps
>  Labels: project
>
> There are two ways to implement "mirroring" (i.e. replicating a topic from 
> one cluster to another):
> 1. Do a simple read from the source and write to the destination with no 
> attempt to maintain the same partitioning or offsets in the destination 
> cluster. In this case the destination cluster may have a different number of 
> partitions, and you can even read from many clusters to create a merged 
> cluster. This flexibility is nice. The downside is that since the 
> partitioning and offsets are not the same a consumer of the source cluster 
> has no equivalent position in the destination cluster. This is the style of 
> mirroring we have implemented in the mirror-maker tool and use for datacenter 
> replication today.
> 2. The second style of replication only would allow creating an exact replica 
> of a source cluster (i.e. all partitions and offsets exactly the same). The 
> nice thing about this is that the offsets and partitions would match exactly. 
> The downside is that it is not possible to merge multiple source clusters 
> this way or have different partitioning. We do not currently support this in 
> mirror maker.
> It would be nice to implement the second style as an option in mirror maker 
> as having an exact replica would be a nice option to have in the case where 
> you are replicating a single cluster only.
> There are some nuances: In order to maintain the exact offsets it is 
> important to guarantee that the producer never resends a message or loses a 
> message. As a result it would be important to have only a single producer for 
> each destination partition, and check the last produced message on startup 
> (using the getOffsets api) so that in the case of a hard crash messages that 
> are re-consumed are not re-emitted.



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


Re: [DISCUSS] KIP-36 - Rack aware replica assignment

2015-10-13 Thread Allen Wang
We discussed the KIP in the hangout today. The recommendation is to make
rack as a broker property in ZooKeeper. For users with existing rack
information stored somewhere, they would need to retrieve the information
at broker start up and dynamically set the rack property, which can be
implemented as a wrapper to bootstrap broker. There will be no interface or
pluggable implementation to retrieve the rack information.

The assumption is that you always need to restart the broker to make a
change to the rack.

Once the rack becomes a broker property, it will be possible to make rack
part of the meta data to help the consumer choose which in sync replica to
consume from as part of the future consumer enhancement.

I will update the KIP.

Thanks,
Allen


On Thu, Oct 8, 2015 at 9:23 AM, Allen Wang  wrote:

> I attended Tuesday's KIP hangout but this KIP was not discussed due to
> time constraint.
>
> However, after hearing discussion of KIP-35, I have the feeling that
> incompatibility (caused by new broker property) between brokers with
> different versions  will be solved there. In addition, having stack in
> broker property as meta data may also help consumers in the future. So I am
> open to adding stack property to broker.
>
> Hopefully we can discuss this in the next KIP hangout.
>
> On Wed, Sep 30, 2015 at 2:46 PM, Allen Wang  wrote:
>
>> Can you send me the information on the next KIP hangout?
>>
>> Currently the broker-rack mapping is not cached. In KafkaApis,
>> RackLocator.getRackInfo() is called each time the mapping is needed for
>> auto topic creation. This will ensure latest mapping is used at any time.
>>
>> The ability to get the complete mapping makes it simple to reuse the same
>> interface in command line tools.
>>
>>
>> On Wed, Sep 30, 2015 at 11:01 AM, Aditya Auradkar <
>> aaurad...@linkedin.com.invalid> wrote:
>>
>>> Perhaps we discuss this during the next KIP hangout?
>>>
>>> I do see that a pluggable rack locator can be useful but I do see a few
>>> concerns:
>>>
>>> - The RackLocator (as described in the document), implies that it can
>>> discover rack information for any node in the cluster. How does it deal
>>> with rack location changes? For example, if I moved broker id (1) from
>>> rack
>>> X to Y, I only have to start that broker with a newer rack config. If
>>> RackLocator discovers broker -> rack information at start up time, any
>>> change to a broker will require bouncing the entire cluster since
>>> createTopic requests can be sent to any node in the cluster.
>>> For this reason it may be simpler to have each node be aware of its own
>>> rack and persist it in ZK during start up time.
>>>
>>> - A pluggable RackLocator relies on an external service being available
>>> to
>>> serve rack information.
>>>
>>> Out of curiosity, I looked up how a couple of other systems deal with
>>> zone/rack awareness.
>>> For Cassandra some interesting modes are:
>>> (Property File configuration)
>>>
>>> http://docs.datastax.com/en/cassandra/2.0/cassandra/architecture/architectureSnitchPFSnitch_t.html
>>> (Dynamic inference)
>>>
>>> http://docs.datastax.com/en/cassandra/2.0/cassandra/architecture/architectureSnitchRackInf_c.html
>>>
>>> Voldemort does a static node -> zone assignment based on configuration.
>>>
>>> Aditya
>>>
>>> On Wed, Sep 30, 2015 at 10:05 AM, Allen Wang 
>>> wrote:
>>>
>>> > I would like to see if we can do both:
>>> >
>>> > - Make RackLocator pluggable to facilitate migration with existing
>>> > broker-rack mapping
>>> >
>>> > - Make rack an optional property for broker. If rack is available from
>>> > broker, treat it as source of truth. For users with existing
>>> broker-rack
>>> > mapping somewhere else, they can use the pluggable way or they can
>>> transfer
>>> > the mapping to the broker rack property.
>>> >
>>> > One thing I am not sure is what happens at rolling upgrade when we have
>>> > rack as a broker property. For brokers with older version of Kafka,
>>> will it
>>> > cause problem for them? If so, is there any workaround? I also think it
>>> > would be better not to have rack in the controller wire protocol but
>>> not
>>> > sure if it is achievable.
>>> >
>>> > Thanks,
>>> > Allen
>>> >
>>> >
>>> >
>>> >
>>> >
>>> > On Mon, Sep 28, 2015 at 4:55 PM, Todd Palino 
>>> wrote:
>>> >
>>> > > I tend to like the idea of a pluggable locator. For example, we
>>> already
>>> > > have an interface for discovering information about the physical
>>> location
>>> > > of servers. I don't relish the idea of having to maintain data in
>>> > multiple
>>> > > places.
>>> > >
>>> > > -Todd
>>> > >
>>> > > On Mon, Sep 28, 2015 at 4:48 PM, Aditya Auradkar <
>>> > > aaurad...@linkedin.com.invalid> wrote:
>>> > >
>>> > > > Thanks for starting this KIP Allen.
>>> > > >
>>> > > > I agree with Gwen that having a RackLocator class that is pluggable
>>> > seems
>>> > > > to be too complex. The KIP refers to potentially non-ZK storage
>>> for the
>>> > > > rack info which I don't think i

[jira] [Assigned] (KAFKA-2644) Run relevant ducktape tests with SASL_PLAIN and SASL_SSL

2015-10-13 Thread Rajini Sivaram (JIRA)

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

Rajini Sivaram reassigned KAFKA-2644:
-

Assignee: Rajini Sivaram

> Run relevant ducktape tests with SASL_PLAIN and SASL_SSL
> 
>
> Key: KAFKA-2644
> URL: https://issues.apache.org/jira/browse/KAFKA-2644
> Project: Kafka
>  Issue Type: Sub-task
>  Components: security
>Reporter: Ismael Juma
>Assignee: Rajini Sivaram
>Priority: Critical
> Fix For: 0.9.0.0
>
>
> We need to define which of the existing ducktape tests are relevant. cc 
> [~rsivaram]



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


[jira] [Commented] (KAFKA-2527) System Test for Quotas in Ducktape

2015-10-13 Thread Aditya Auradkar (JIRA)

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

Aditya Auradkar commented on KAFKA-2527:


[~gwenshap] - Thanks!

> System Test for Quotas in Ducktape
> --
>
> Key: KAFKA-2527
> URL: https://issues.apache.org/jira/browse/KAFKA-2527
> Project: Kafka
>  Issue Type: Sub-task
>Reporter: Dong Lin
>Assignee: Dong Lin
>  Labels: quota
> Fix For: 0.9.0.0
>
>




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


[GitHub] kafka pull request: MINOR: flush record collector after local stat...

2015-10-13 Thread ymatsuda
GitHub user ymatsuda opened a pull request:

https://github.com/apache/kafka/pull/304

MINOR: flush record collector after local state flush

@guozhangwang 
Fix the order of flushing. Undoing the change I did sometime ago.

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/ymatsuda/kafka flush_order

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/304.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #304


commit e9238c31071426a3d82435415fbb3a31a9ca2ae8
Author: Yasuhiro Matsuda 
Date:   2015-10-13T21:43:04Z

MINOR: flush record collector after local state flush




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


Build failed in Jenkins: kafka-trunk-jdk7 #683

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[cshapi] KAFKA-2527; System Test for Quotas in Ducktape

--
Started by an SCM change
[EnvInject] - Loading node environment variables.
Building remotely on ubuntu-5 (docker Ubuntu ubuntu5 ubuntu) in workspace 

 > git rev-parse --is-inside-work-tree # timeout=10
Fetching changes from the remote Git repository
 > git config remote.origin.url 
 > https://git-wip-us.apache.org/repos/asf/kafka.git # timeout=10
Fetching upstream changes from https://git-wip-us.apache.org/repos/asf/kafka.git
 > git --version # timeout=10
 > git -c core.askpass=true fetch --tags --progress 
 > https://git-wip-us.apache.org/repos/asf/kafka.git 
 > +refs/heads/*:refs/remotes/origin/*
 > git rev-parse refs/remotes/origin/trunk^{commit} # timeout=10
 > git rev-parse refs/remotes/origin/origin/trunk^{commit} # timeout=10
Checking out Revision 123d27e4d005a384611671133fbecde7e390d24f 
(refs/remotes/origin/trunk)
 > git config core.sparsecheckout # timeout=10
 > git checkout -f 123d27e4d005a384611671133fbecde7e390d24f
 > git rev-list 36d4469326fe20c3f0657315321e6ad515530a3e # timeout=10
Setting 
GRADLE_2_1_HOME=/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.GradleInstallation/Gradle_2.1
Setting JDK_1_7_LATEST__HOME=/home/jenkins/tools/java/latest1.7
[kafka-trunk-jdk7] $ /bin/bash -xe /tmp/hudson5294160929450402833.sh
+ 
/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.GradleInstallation/Gradle_2.1/bin/gradle
To honour the JVM settings for this build a new JVM will be forked. Please 
consider using the daemon: 
http://gradle.org/docs/2.1/userguide/gradle_daemon.html.
Building project 'core' with Scala version 2.10.5
:downloadWrapper UP-TO-DATE

BUILD SUCCESSFUL

Total time: 18.766 secs
Setting 
GRADLE_2_1_HOME=/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.GradleInstallation/Gradle_2.1
Setting JDK_1_7_LATEST__HOME=/home/jenkins/tools/java/latest1.7
[kafka-trunk-jdk7] $ /bin/bash -xe /tmp/hudson6445953935321121713.sh
+ ./gradlew -Dorg.gradle.project.maxParallelForks=1 clean jarAll docsJarAll 
testAll
To honour the JVM settings for this build a new JVM will be forked. Please 
consider using the daemon: 
http://gradle.org/docs/2.4/userguide/gradle_daemon.html.
Building project 'core' with Scala version 2.10.5
:clean UP-TO-DATE
:clients:clean
:contrib:clean UP-TO-DATE
:copycat:clean UP-TO-DATE
:core:clean
:examples:clean
:log4j-appender:clean
:streams:clean
:tools:clean
:contrib:hadoop-consumer:clean
:contrib:hadoop-producer:clean
:copycat:api:clean
:copycat:file:clean
:copycat:json:clean
:copycat:runtime:clean
:jar_core_2_10_5
Building project 'core' with Scala version 2.10.5
:kafka-trunk-jdk7:clients:compileJavaNote: 

 uses or overrides a deprecated API.
Note: Recompile with -Xlint:deprecation for details.
Note: Some input files use unchecked or unsafe operations.
Note: Recompile with -Xlint:unchecked for details.

:kafka-trunk-jdk7:clients:processResources UP-TO-DATE
:kafka-trunk-jdk7:clients:classes
:kafka-trunk-jdk7:clients:determineCommitId UP-TO-DATE
:kafka-trunk-jdk7:clients:createVersionFile
:kafka-trunk-jdk7:clients:jar
:kafka-trunk-jdk7:log4j-appender:compileJava
:kafka-trunk-jdk7:log4j-appender:processResources UP-TO-DATE
:kafka-trunk-jdk7:log4j-appender:classes
:kafka-trunk-jdk7:log4j-appender:jar
:kafka-trunk-jdk7:core:compileJava UP-TO-DATE
:kafka-trunk-jdk7:core:compileScala
:78:
 value DEFAULT_TIMESTAMP in object OffsetCommitRequest is deprecated: see 
corresponding Javadoc for more information.

org.apache.kafka.common.requests.OffsetCommitRequest.DEFAULT_TIMESTAMP
 ^
:36:
 value DEFAULT_TIMESTAMP in object OffsetCommitRequest is deprecated: see 
corresponding Javadoc for more information.
 commitTimestamp: Long = 
org.apache.kafka.common.requests.OffsetCommitRequest.DEFAULT_TIMESTAMP,

  ^
:37:
 value DEFAULT_TIMESTAMP in object OffsetCommitRequest is deprecated: see 
corresponding Javadoc for more information.
 expireTimestamp: Long = 
org.apache.kafka.common.requests.OffsetCommitRequest.DEFAULT_TIMESTAMP) {

  ^


Re: [DISCUSS] KIP-32 - Add CreateTime and LogAppendTime to Kafka message

2015-10-13 Thread Guozhang Wang
Just to complete Jay's option, here is my understanding:

1. For log retention: if we want to remove data before time t, we look into
the index file of each segment and find the largest timestamp t' < t, find
the corresponding timestamp and start scanning to the end of the segment,
if there is no entry with timestamp >= t, we can delete this segment; if a
segment's index smallest timestamp is larger than t, we can skip that
segment.

2. For log rolling: if we want to start a new segment after time t, we look
into the active segment's index file, if the largest timestamp is already >
t, we can roll a new segment immediately; if it is < t, we read its
corresponding offset and start scanning to the end of the segment, if we
find a record whose timestamp > t, we can roll a new segment.

For log rolling we only need to possibly scan a small portion the active
segment, which should be fine; for log retention we may in the worst case
end up scanning all segments, but in practice we may skip most of them
since their smallest timestamp in the index file is larger than t.

Guozhang


On Tue, Oct 13, 2015 at 12:52 AM, Jay Kreps  wrote:

> I think it should be possible to index out-of-order timestamps. The
> timestamp index would be similar to the offset index, a memory mapped file
> appended to as part of the log append, but would have the format
>   timestamp offset
> The timestamp entries would be monotonic and as with the offset index would
> be no more often then every 4k (or some configurable threshold to keep the
> index small--actually for timestamp it could probably be much more sparse
> than 4k).
>
> A search for a timestamp t yields an offset o before which no prior message
> has a timestamp >= t. In other words if you read the log starting with o
> you are guaranteed not to miss any messages occurring at t or later though
> you may get many before t (due to out-of-orderness). Unlike the offset
> index this bound doesn't really have to be tight (i.e. probably no need to
> go search the log itself, though you could).
>
> -Jay
>
> On Tue, Oct 13, 2015 at 12:32 AM, Jay Kreps  wrote:
>
> > Here's my basic take:
> > - I agree it would be nice to have a notion of time baked in if it were
> > done right
> > - All the proposals so far seem pretty complex--I think they might make
> > things worse rather than better overall
> > - I think adding 2x8 byte timestamps to the message is probably a
> > non-starter from a size perspective
> > - Even if it isn't in the message, having two notions of time that
> control
> > different things is a bit confusing
> > - The mechanics of basing retention etc on log append time when that's
> not
> > in the log seem complicated
> >
> > To that end here is a possible 4th option. Let me know what you think.
> >
> > The basic idea is that the message creation time is closest to what the
> > user actually cares about but is dangerous if set wrong. So rather than
> > substitute another notion of time, let's try to ensure the correctness of
> > message creation time by preventing arbitrarily bad message creation
> times.
> >
> > First, let's see if we can agree that log append time is not something
> > anyone really cares about but rather an implementation detail. The
> > timestamp that matters to the user is when the message occurred (the
> > creation time). The log append time is basically just an approximation to
> > this on the assumption that the message creation and the message receive
> on
> > the server occur pretty close together and the reason to prefer .
> >
> > But as these values diverge the issue starts to become apparent. Say you
> > set the retention to one week and then mirror data from a topic
> containing
> > two years of retention. Your intention is clearly to keep the last week,
> > but because the mirroring is appending right now you will keep two years.
> >
> > The reason we are liking log append time is because we are (justifiably)
> > concerned that in certain situations the creation time may not be
> > trustworthy. This same problem exists on the servers but there are fewer
> > servers and they just run the kafka code so it is less of an issue.
> >
> > There are two possible ways to handle this:
> >
> >1. Just tell people to add size based retention. I think this is not
> >entirely unreasonable, we're basically saying we retain data based on
> the
> >timestamp you give us in the data. If you give us bad data we will
> retain
> >it for a bad amount of time. If you want to ensure we don't retain
> "too
> >much" data, define "too much" by setting a time-based retention
> setting.
> >This is not entirely unreasonable but kind of suffers from a "one bad
> >apple" problem in a very large environment.
> >2. Prevent bad timestamps. In general we can't say a timestamp is bad.
> >However the definition we're implicitly using is that we think there
> are a
> >set of topics/clusters where the creation timestamp should always be
> "very

[GitHub] kafka pull request: MINOR: flush record collector after local stat...

2015-10-13 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/304


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[GitHub] kafka pull request: KAFKA-2536: topics tool should allow users to ...

2015-10-13 Thread granthenke
GitHub user granthenke opened a pull request:

https://github.com/apache/kafka/pull/305

KAFKA-2536: topics tool should allow users to alter topic configuration

This is a minimal revert of some backward incompatible changes made in 
KAFKA-2205, with the addition of the deprecation logging message. 

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/granthenke/kafka topic-configs

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/305.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #305


commit d07df092c83b212e5fd5a061e75fc57cceef003a
Author: Grant Henke 
Date:   2015-10-13T22:38:42Z

KAFKA-2536: topics tool should allow users to alter topic configuration




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2536:
---

GitHub user granthenke opened a pull request:

https://github.com/apache/kafka/pull/305

KAFKA-2536: topics tool should allow users to alter topic configuration

This is a minimal revert of some backward incompatible changes made in 
KAFKA-2205, with the addition of the deprecation logging message. 

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/granthenke/kafka topic-configs

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/305.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #305


commit d07df092c83b212e5fd5a061e75fc57cceef003a
Author: Grant Henke 
Date:   2015-10-13T22:38:42Z

KAFKA-2536: topics tool should allow users to alter topic configuration




> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>Reporter: Gwen Shapira
>Assignee: Grant Henke
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


[jira] [Updated] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke updated KAFKA-2536:
---
Status: Patch Available  (was: Open)

> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>  Components: tools
>Reporter: Gwen Shapira
>Assignee: Grant Henke
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


[jira] [Updated] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke updated KAFKA-2536:
---
Component/s: tools

> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>  Components: tools
>Reporter: Gwen Shapira
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


[jira] [Updated] (KAFKA-2536) topics tool should allow users to alter topic configuration

2015-10-13 Thread Grant Henke (JIRA)

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

Grant Henke updated KAFKA-2536:
---
Fix Version/s: 0.9.0.0

> topics tool should allow users to alter topic configuration
> ---
>
> Key: KAFKA-2536
> URL: https://issues.apache.org/jira/browse/KAFKA-2536
> Project: Kafka
>  Issue Type: Bug
>  Components: tools
>Reporter: Gwen Shapira
>Assignee: Grant Henke
> Fix For: 0.9.0.0
>
>
> When we added dynamic config, we added a kafka-config tool (which can be used 
> to maintain configs for non-topic entities), and remove the capability from 
> kafka-topic tool.
> Removing the capability from kafka-topic is:
> 1. Breaks backward compatibility in our most essential tools. This has 
> significant impact on usability.
> 2. Kinda confusing that --create --config works but --alter --config does 
> not. 
> I suggest fixing this.



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


Build failed in Jenkins: kafka-trunk-jdk8 #25

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[cshapi] KAFKA-2527; System Test for Quotas in Ducktape

--
[...truncated 6337 lines...]
org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > 
testSourceTasksStdin PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > testTaskClass 
PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > 
testMultipleSourcesInvalid PASSED

org.apache.kafka.copycat.file.FileStreamSinkConnectorTest > testSinkTasks PASSED

org.apache.kafka.copycat.file.FileStreamSinkConnectorTest > testTaskClass PASSED

org.apache.kafka.copycat.file.FileStreamSinkTaskTest > testPutFlush PASSED
:copycat:json:checkstyleMain
:copycat:json:compileTestJavawarning: [options] bootstrap class path not set in 
conjunction with -source 1.7
Note: 

 uses unchecked or unsafe operations.
Note: Recompile with -Xlint:unchecked for details.
1 warning

:copycat:json:processTestResources UP-TO-DATE
:copycat:json:testClasses
:copycat:json:checkstyleTest
:copycat:json:test

org.apache.kafka.copycat.json.JsonConverterTest > longToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCacheSchemaToJsonConversion PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
nullSchemaAndMapNonStringKeysToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > floatToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > booleanToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndMapToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > stringToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timestampToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCopycatSchemaMetadataTranslation PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timestampToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > decimalToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToCopycatStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToJsonNonStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > longToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mismatchSchemaJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCacheSchemaToCopycatConversion PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testJsonSchemaMetadataTranslation PASSED

org.apache.kafka.copycat.json.JsonConverterTest > bytesToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > shortToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > intToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > structToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > stringToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndArrayToJson 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > byteToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaPrimitiveToCopycat 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > byteToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > intToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > dateToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > noSchemaToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > noSchemaToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndPrimitiveToJson 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToJsonStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > arrayToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timeToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > structToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > shortToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > dateToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > doubleToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timeToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > floatToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > decimalToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > arrayToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > booleanToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToCopycatNonStringKeys 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > bytesToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > doubleToCopycat PASSED
:copycat:runtime:checkstyleMain
:copycat:runtime:compileTestJavawarning: [options] bootstrap class path not set 
in conjunction with -source 1.7
Note: Some input files use unchecked or unsafe ope

Build failed in Jenkins: kafka-trunk-jdk7 #684

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[wangguoz] MINOR: flush record collector after local state flush

--
Started by an SCM change
[EnvInject] - Loading node environment variables.
Building remotely on ubuntu-6 (docker Ubuntu ubuntu) in workspace 

Cloning the remote Git repository
Cloning repository https://git-wip-us.apache.org/repos/asf/kafka.git
 > git init  # timeout=10
Fetching upstream changes from https://git-wip-us.apache.org/repos/asf/kafka.git
 > git --version # timeout=10
 > git -c core.askpass=true fetch --tags --progress 
 > https://git-wip-us.apache.org/repos/asf/kafka.git 
 > +refs/heads/*:refs/remotes/origin/*
 > git config remote.origin.url 
 > https://git-wip-us.apache.org/repos/asf/kafka.git # timeout=10
 > git config --add remote.origin.fetch +refs/heads/*:refs/remotes/origin/* # 
 > timeout=10
 > git config remote.origin.url 
 > https://git-wip-us.apache.org/repos/asf/kafka.git # timeout=10
Fetching upstream changes from https://git-wip-us.apache.org/repos/asf/kafka.git
 > git -c core.askpass=true fetch --tags --progress 
 > https://git-wip-us.apache.org/repos/asf/kafka.git 
 > +refs/heads/*:refs/remotes/origin/*
 > git rev-parse refs/remotes/origin/trunk^{commit} # timeout=10
 > git rev-parse refs/remotes/origin/origin/trunk^{commit} # timeout=10
Checking out Revision b1ce9494e3a964613d3d9534471df79593514c77 
(refs/remotes/origin/trunk)
 > git config core.sparsecheckout # timeout=10
 > git checkout -f b1ce9494e3a964613d3d9534471df79593514c77
 > git rev-list 123d27e4d005a384611671133fbecde7e390d24f # timeout=10
Setting 
GRADLE_2_1_HOME=/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.GradleInstallation/Gradle_2.1
Setting JDK_1_7_LATEST__HOME=/home/jenkins/tools/java/latest1.7
[kafka-trunk-jdk7] $ /bin/bash -xe /tmp/hudson5263761382958751437.sh
+ 
/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.GradleInstallation/Gradle_2.1/bin/gradle
To honour the JVM settings for this build a new JVM will be forked. Please 
consider using the daemon: 
http://gradle.org/docs/2.1/userguide/gradle_daemon.html.

FAILURE: Build failed with an exception.

* What went wrong:
Unable to start the daemon process.
This problem might be caused by incorrect configuration of the daemon.
For example, an unrecognized jvm option is used.
Please refer to the user guide chapter on the daemon at 
http://gradle.org/docs/2.1/userguide/gradle_daemon.html
Please read below process output to find out more:
---
22:55:09.927 [main] DEBUG o.g.l.daemon.bootstrap.DaemonMain - Assuming the 
daemon was started with following jvm opts: [-XX:MaxPermSize=512m, -Xss2m, 
-Xmx1024m, -Dfile.encoding=ISO-8859-1, -Duser.country=US, -Duser.language=en, 
-Duser.variant]
22:55:10.697 [main] DEBUG o.g.l.daemon.server.DaemonServices - Creating daemon 
context with opts: [-XX:MaxPermSize=512m, -Xss2m, -Xmx1024m, 
-Dfile.encoding=ISO-8859-1, -Duser.country=US, -Duser.language=en, 
-Duser.variant]
22:55:10.776 [INFO] [org.gradle.launcher.daemon.server.Daemon] start() called 
on daemon - 
DefaultDaemonContext[uid=dadf1ed9-bc2a-47a8-bb8c-100a6eb80f76,javaHome=/home/jenkins/tools/java/jdk1.7.0_25-32,daemonRegistryDir=/home/jenkins/.gradle/daemon,pid=5092,idleTimeout=12,daemonOpts=-XX:MaxPermSize=512m,-Xss2m,-Xmx1024m,-Dfile.encoding=ISO-8859-1,-Duser.country=US,-Duser.language=en,-Duser.variant]
22:55:10.782 [DEBUG] [org.gradle.launcher.daemon.server.DaemonStateCoordinator] 
updating lastActivityAt to 1444776910782
#
# A fatal error has been detected by the Java Runtime Environment:
#
#  SIGSEGV (0xb) at pc=0x9256d52c, pid=5092, tid=4136651584
#
# JRE version: 7.0_25-b15
# Java VM: Java HotSpot(TM) Server VM (23.25-b01 mixed mode linux-x86 )
# Problematic frame:
# C  [libnet.so+0x352c]  _init+0x620
#
# Failed to write core dump. Core dumps have been disabled. To enable core 
dumping, try "ulimit -c unlimited" before starting Java again
#
# An error report file with more information is saved as:
# /home/jenkins/.gradle/daemon/2.1/hs_err_pid5092.log
#
# If you would like to submit a bug report, please visit:
#   http://bugreport.sun.com/bugreport/crash.jsp
# The crash happened outside the Java Virtual Machine in native code.
# See problematic frame for where to report the bug.
#


* Try:
Run with --stacktrace option to get the stack trace. Run with --info or --debug 
option to get more log output.
Build step 'Execute shell' marked build as failure
Recording test results
Setting 
GRADLE_2_1_HOME=/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.GradleInstallation/Gradle_2.1
Setting JDK_1_7_LATEST__HOME=/home/jenkins/tools/java/latest1.7
ERROR: Publisher 'Publish JUnit test result report' failed: No test report 
files were found. Configuration error?
Setting 
GRADLE_2_1_HOME=/home/jenkins/jenkins-slave/tools/hudson.plugins.gradle.G

Build failed in Jenkins: kafka-trunk-jdk8 #26

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[wangguoz] MINOR: flush record collector after local state flush

--
[...truncated 6015 lines...]
org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > 
testSourceTasksStdin PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > testTaskClass 
PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > 
testMultipleSourcesInvalid PASSED

org.apache.kafka.copycat.file.FileStreamSinkConnectorTest > testSinkTasks PASSED

org.apache.kafka.copycat.file.FileStreamSinkConnectorTest > testTaskClass PASSED

org.apache.kafka.copycat.file.FileStreamSinkTaskTest > testPutFlush PASSED
:copycat:json:checkstyleMain
:copycat:json:compileTestJavawarning: [options] bootstrap class path not set in 
conjunction with -source 1.7
Note: 

 uses unchecked or unsafe operations.
Note: Recompile with -Xlint:unchecked for details.
1 warning

:copycat:json:processTestResources UP-TO-DATE
:copycat:json:testClasses
:copycat:json:checkstyleTest
:copycat:json:test

org.apache.kafka.copycat.json.JsonConverterTest > longToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCacheSchemaToJsonConversion PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
nullSchemaAndMapNonStringKeysToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > floatToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > booleanToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndMapToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > stringToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timestampToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCopycatSchemaMetadataTranslation PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timestampToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > decimalToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToCopycatStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToJsonNonStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > longToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mismatchSchemaJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCacheSchemaToCopycatConversion PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testJsonSchemaMetadataTranslation PASSED

org.apache.kafka.copycat.json.JsonConverterTest > bytesToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > shortToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > intToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > structToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > stringToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndArrayToJson 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > byteToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaPrimitiveToCopycat 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > byteToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > intToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > dateToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > noSchemaToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > noSchemaToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndPrimitiveToJson 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToJsonStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > arrayToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timeToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > structToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > shortToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > dateToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > doubleToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timeToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > floatToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > decimalToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > arrayToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > booleanToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToCopycatNonStringKeys 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > bytesToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > doubleToCopycat PASSED
:copycat:runtime:checkstyleMain
:copycat:runtime:compileTestJavawarning: [options] bootstrap class path not set 
in conjunction with -source 1.7
Note: Some input files use unchecked or u

[GitHub] kafka pull request: MINOR: ignore wakeups when committing offsets ...

2015-10-13 Thread hachikuji
GitHub user hachikuji opened a pull request:

https://github.com/apache/kafka/pull/306

MINOR: ignore wakeups when committing offsets on consumer close



You can merge this pull request into a Git repository by running:

$ git pull https://github.com/hachikuji/kafka 
handle-wakeup-in-consumer-close

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/306.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #306


commit 951bba273871ca9c684340e7e07fd915ea8e1fce
Author: Jason Gustafson 
Date:   2015-10-13T23:56:50Z

MINOR: ignore wakeups when committing offsets on consumer close




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2017) Persist Coordinator State for Coordinator Failover

2015-10-13 Thread Guozhang Wang (JIRA)

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

Guozhang Wang commented on KAFKA-2017:
--

[~hachikuji] [~onurkaraman] [~junrao] With the new protocol, coordinator does 
not need to remember any member metadata except the member ids since now we 
only validate on member-id and generation-id. So after KAFKA-2464 is merged in 
I propose to store the group metadata as:

{code}
/coordinator/consumers/[groupId]:
version: short
generationId: int
members: String   // <- member-ids split by "," and do now allow "," in 
member-id, the first member is always the leader.
{code}

The reading logic is: 

1. Upon handling HB / OffsetCommit / OffsetFetch request, after validating the 
group belongs to itself and coordinator.isActive, if the group does not exist 
in the group metadata cache, try reading from ZK; leave other non-persistent 
fields in the GroupMetadata and MemberMetadata as null.

2. Upon handling JoinGroup, after validating the group belongs to itself and 
coordinator.isActive, if the group does not exist in the group metadata cache, 
try reading from ZK; if the consumer already exists, follow the normal path of 
handlJoinGroup, the only difference is that we will update the member metadata 
and always trigger a rebalance.

3. Upon handling SyncGroup, after validating the group belongs to itself and 
coordinator.isActive, if the group does not exist in the group metadata cache, 
try reading from ZK; then follow the normal path of handleSyncGroup.

The write logic is as follows:

After the Join-group barrier, update the ZK with the generation id / leader-id 
/ members.

With this proposal, we do not need a "Initialize" state as in the original 
proposal 
https://cwiki.apache.org/confluence/display/KAFKA/Kafka+Client-side+Assignment+Proposal.

> Persist Coordinator State for Coordinator Failover
> --
>
> Key: KAFKA-2017
> URL: https://issues.apache.org/jira/browse/KAFKA-2017
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Affects Versions: 0.9.0.0
>Reporter: Onur Karaman
>Assignee: Guozhang Wang
> Fix For: 0.9.0.0
>
> Attachments: KAFKA-2017.patch, KAFKA-2017_2015-05-20_09:13:39.patch, 
> KAFKA-2017_2015-05-21_19:02:47.patch
>
>
> When a coordinator fails, the group membership protocol tries to failover to 
> a new coordinator without forcing all the consumers rejoin their groups. This 
> is possible if the coordinator persists its state so that the state can be 
> transferred during coordinator failover. This state consists of most of the 
> information in GroupRegistry and ConsumerRegistry.



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


[jira] [Commented] (KAFKA-2017) Persist Coordinator State for Coordinator Failover

2015-10-13 Thread Jason Gustafson (JIRA)

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

Jason Gustafson commented on KAFKA-2017:


[~guozhang] I think that can work. The only weird case is 3. Since the 
coordinator won't have the assignments from the previous generation, it will 
generally have to force the group to rebalance, right? This is basically the 
case where the coordinator fails before all members have collected their 
assignment, but after the group state has been written to Zookeeper. This 
should be a rare case, so I think it's reasonable to have to rebalance when it 
happens.

> Persist Coordinator State for Coordinator Failover
> --
>
> Key: KAFKA-2017
> URL: https://issues.apache.org/jira/browse/KAFKA-2017
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Affects Versions: 0.9.0.0
>Reporter: Onur Karaman
>Assignee: Guozhang Wang
> Fix For: 0.9.0.0
>
> Attachments: KAFKA-2017.patch, KAFKA-2017_2015-05-20_09:13:39.patch, 
> KAFKA-2017_2015-05-21_19:02:47.patch
>
>
> When a coordinator fails, the group membership protocol tries to failover to 
> a new coordinator without forcing all the consumers rejoin their groups. This 
> is possible if the coordinator persists its state so that the state can be 
> transferred during coordinator failover. This state consists of most of the 
> information in GroupRegistry and ConsumerRegistry.



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


Re: KIP-28 does not allow Processor to specify partition of output message

2015-10-13 Thread Guozhang Wang
Hi Randall,

You can try to set the partitioner class as
ProducerConfig.PARTITIONER_CLASS_CONFIG in the StreamsConfig, its interface
can be found in

org.apache.kafka.clients.producer.Partitioner

Let me know if it works for you.

Guozhang

On Tue, Oct 13, 2015 at 10:59 AM, Randall Hauch  wrote:

> The new streams API added with KIP-28 is great. I’ve been using it on a
> prototype for a few weeks, and I’m looking forward to it being included in
> 0.9.0. However, at the moment, a Processor implementation is not able to
> specify the partition number when it outputs messages.
>
> I’d be happy to log a JIRA and create a PR to add it to the API, but
> without knowing all of the history I’m wondering if leaving it out of the
> API was intentional.
>
> Thoughts?
>
> Best regards,
>
> Randall Hauch
>



-- 
-- Guozhang


[jira] [Commented] (KAFKA-2484) Add schema projection utilities

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2484:
---

GitHub user Ishiihara opened a pull request:

https://github.com/apache/kafka/pull/307

KAFKA-2484: Add schema projection utilities

This PR adds schema projection utilities to copycat. 

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/Ishiihara/kafka schema-projection

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/307.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #307


commit 9747208fead30dcaed5295f674650b61cce25977
Author: Liquan Pei 
Date:   2015-10-14T01:16:32Z

Add schema projection utilities




> Add schema projection utilities
> ---
>
> Key: KAFKA-2484
> URL: https://issues.apache.org/jira/browse/KAFKA-2484
> Project: Kafka
>  Issue Type: Sub-task
>  Components: copycat
>Reporter: Ewen Cheslack-Postava
>Assignee: Liquan Pei
>Priority: Minor
> Fix For: 0.9.0.0
>
>
> Since Copycat has support for versioned schemas and connectors may encounter 
> different versions of the same schema, it will be useful for some connectors 
> to be able to project between different versions of a schema, or have an 
> automatic way to try to project to a target schema (e.g. an existing database 
> table the connector is trying to write data to).
> These utilities should be pretty small because the complex types we support 
> are fairly limited. The primary code required will be for Structs. However, 
> we should take care in designing these utilities since there may be 
> performance implications. For example, when projecting between two schemas, 
> it would be better to come up with a plan object that can efficiently perform 
> the project and be able to reuse that plan many times.



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


[GitHub] kafka pull request: KAFKA-2484: Add schema projection utilities

2015-10-13 Thread Ishiihara
GitHub user Ishiihara opened a pull request:

https://github.com/apache/kafka/pull/307

KAFKA-2484: Add schema projection utilities

This PR adds schema projection utilities to copycat. 

You can merge this pull request into a Git repository by running:

$ git pull https://github.com/Ishiihara/kafka schema-projection

Alternatively you can review and apply these changes as the patch at:

https://github.com/apache/kafka/pull/307.patch

To close this pull request, make a commit to your master/trunk branch
with (at least) the following in the commit message:

This closes #307


commit 9747208fead30dcaed5295f674650b61cce25977
Author: Liquan Pei 
Date:   2015-10-14T01:16:32Z

Add schema projection utilities




---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


Re: KIP-28 does not allow Processor to specify partition of output message

2015-10-13 Thread Randall Hauch
This overrides the partitioning logic for all topics, right? That means I have 
to explicitly call the default partitioning logic for all topics except those 
that my Producer forwards. I’m guess the best way to do by extending 
org.apache.kafka.clients.producer.DefaultProducer. Of course, with multiple 
sinks in my topology, I have to put all of the partitioning logic inside a 
single class.

What would you think about adding an overloaded TopologyBuilder.addSink(…) 
method that takes a Partitioner (or better yet a smaller functional interface). 
The resulting SinkProcessor could use that Partitioner instance to set the 
partition number? That’d be super convenient for users, would keep the logic 
where it belongs (where the topology defines the sinks), and best of all the 
implementations won't have to worry about any other topics, such as those used 
by stores, metrics, or other sinks.

Best regards,

Randall


On October 13, 2015 at 8:09:41 PM, Guozhang Wang (wangg...@gmail.com) wrote:

Hi Randall,  

You can try to set the partitioner class as  
ProducerConfig.PARTITIONER_CLASS_CONFIG in the StreamsConfig, its interface  
can be found in  

org.apache.kafka.clients.producer.Partitioner  

Let me know if it works for you.  

Guozhang  

On Tue, Oct 13, 2015 at 10:59 AM, Randall Hauch  wrote:  

> The new streams API added with KIP-28 is great. I’ve been using it on a  
> prototype for a few weeks, and I’m looking forward to it being included in  
> 0.9.0. However, at the moment, a Processor implementation is not able to  
> specify the partition number when it outputs messages.  
>  
> I’d be happy to log a JIRA and create a PR to add it to the API, but  
> without knowing all of the history I’m wondering if leaving it out of the  
> API was intentional.  
>  
> Thoughts?  
>  
> Best regards,  
>  
> Randall Hauch  
>  



--  
-- Guozhang  


[jira] [Commented] (KAFKA-2017) Persist Coordinator State for Coordinator Failover

2015-10-13 Thread Joel Koshy (JIRA)

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

Joel Koshy commented on KAFKA-2017:
---

In offline threads we were going to discuss the options of Kafka-based storage 
vs zookeeper for persisting state. I think it would be beneficial to finish 
that discussion either here or in the wiki or mailing list before doing this 
exclusively in ZK for 0.9 right?

> Persist Coordinator State for Coordinator Failover
> --
>
> Key: KAFKA-2017
> URL: https://issues.apache.org/jira/browse/KAFKA-2017
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Affects Versions: 0.9.0.0
>Reporter: Onur Karaman
>Assignee: Guozhang Wang
> Fix For: 0.9.0.0
>
> Attachments: KAFKA-2017.patch, KAFKA-2017_2015-05-20_09:13:39.patch, 
> KAFKA-2017_2015-05-21_19:02:47.patch
>
>
> When a coordinator fails, the group membership protocol tries to failover to 
> a new coordinator without forcing all the consumers rejoin their groups. This 
> is possible if the coordinator persists its state so that the state can be 
> transferred during coordinator failover. This state consists of most of the 
> information in GroupRegistry and ConsumerRegistry.



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


[GitHub] kafka pull request: KAFKA-2632: move fetchable check ahead in hand...

2015-10-13 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/295


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


[jira] [Commented] (KAFKA-2632) Move fetchable check from fetchedRecords to fetch response handler

2015-10-13 Thread ASF GitHub Bot (JIRA)

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

ASF GitHub Bot commented on KAFKA-2632:
---

Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/295


> Move fetchable check from fetchedRecords to fetch response handler
> --
>
> Key: KAFKA-2632
> URL: https://issues.apache.org/jira/browse/KAFKA-2632
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Reporter: Guozhang Wang
>Assignee: Guozhang Wang
> Fix For: 0.9.0.0
>
>
> Since we check if the partition is fetchable or not in fetchedRecords(), 
> there is a bug when the partition is paused during there is in-flight fetch 
> request, it will not be ignored in fetch response handler but after that, in 
> fetchedRecords(), causing the fetcher to update the fetched position already; 
> later no fetch requests will ever be sent to the broker for this partition 
> since consumed != fetched.
> The proposed fix is to move this check from fetchedRecords to 
> handleFetchResponse in Fetcher.



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


[jira] [Resolved] (KAFKA-2632) Move fetchable check from fetchedRecords to fetch response handler

2015-10-13 Thread Gwen Shapira (JIRA)

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

Gwen Shapira resolved KAFKA-2632.
-
Resolution: Fixed

Issue resolved by pull request 295
[https://github.com/apache/kafka/pull/295]

> Move fetchable check from fetchedRecords to fetch response handler
> --
>
> Key: KAFKA-2632
> URL: https://issues.apache.org/jira/browse/KAFKA-2632
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Reporter: Guozhang Wang
>Assignee: Guozhang Wang
> Fix For: 0.9.0.0
>
>
> Since we check if the partition is fetchable or not in fetchedRecords(), 
> there is a bug when the partition is paused during there is in-flight fetch 
> request, it will not be ignored in fetch response handler but after that, in 
> fetchedRecords(), causing the fetcher to update the fetched position already; 
> later no fetch requests will ever be sent to the broker for this partition 
> since consumed != fetched.
> The proposed fix is to move this check from fetchedRecords to 
> handleFetchResponse in Fetcher.



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


[GitHub] kafka pull request: MINOR: ignore wakeups when committing offsets ...

2015-10-13 Thread asfgit
Github user asfgit closed the pull request at:

https://github.com/apache/kafka/pull/306


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastruct...@apache.org or file a JIRA ticket
with INFRA.
---


Re: [DISCUSS] KIP-32 - Add CreateTime and LogAppendTime to Kafka message

2015-10-13 Thread Jiangjie Qin
Hey Jay and Guozhang,

Thanks a lot for the reply. So if I understand correctly, Jay's proposal is:

1. Let client stamp the message create time.
2. Broker build index based on client-stamped message create time.
3. Broker only takes message whose create time is withing current time
plus/minus T (T is a configuration *max.append.delay*, could be topic level
configuration), if the timestamp is out of this range, broker rejects the
message.
4. Because the create time of messages can be out of order, when broker
builds the time based index it only provides the guarantee that if a
consumer starts consuming from the offset returned by searching by
timestamp t, they will not miss any message created after t, but might see
some messages created before t.

To build the time based index, every time when a broker needs to insert a
new time index entry, the entry would be {Largest_Timestamp_Ever_Seen ->
Current_Offset}. This basically means any timestamp larger than the
Largest_Timestamp_Ever_Seen must come after this offset because it never
saw them before. So we don't miss any message with larger timestamp.

(@Guozhang, in this case, for log retention we only need to take a look at
the last time index entry, because it must be the largest timestamp ever,
if that timestamp is overdue, we can safely delete any log segment before
that. So we don't need to scan the log segment file for log retention)

I assume that we are still going to have the new FetchRequest to allow the
time index replication for replicas.

I think Jay's main point here is that we don't want to have two timestamp
concepts in Kafka, which I agree is a reasonable concern. And I also agree
that create time is more meaningful than LogAppendTime for users. But I am
not sure if making everything base on Create Time would work in all cases.
Here are my questions about this approach:

1. Let's say we have two source clusters that are mirrored to the same
target cluster. For some reason one of the mirror maker from a cluster dies
and after fix the issue we want to resume mirroring. In this case it is
possible that when the mirror maker resumes mirroring, the timestamp of the
messages have already gone beyond the acceptable timestamp range on broker.
In order to let those messages go through, we have to bump up the
*max.append.delay
*for all the topics on the target broker. This could be painful.

2. Let's say in the above scenario we let the messages in, at that point
some log segments in the target cluster might have a wide range of
timestamps, like Guozhang mentioned the log rolling could be tricky because
the first time index entry does not necessarily have the smallest timestamp
of all the messages in the log segment. Instead, it is the largest
timestamp ever seen. We have to scan the entire log to find the message
with smallest offset to see if we should roll.

3. Theoretically it is possible that an older log segment contains
timestamps that are older than all the messages in a newer log segment. It
would be weird that we are supposed to delete the newer log segment before
we delete the older log segment.

4. In bootstrap case, if we reload the data to a Kafka cluster, we have to
make sure we configure the topic correctly before we load the data.
Otherwise the message might either be rejected because the timestamp is too
old, or it might be deleted immediately because the retention time has
reached.

I am very concerned about the operational overhead and the ambiguity of
guarantees we introduce if we purely rely on CreateTime.

It looks to me that the biggest issue of adopting CreateTime everywhere is
CreateTime can have big gaps. These gaps could be caused by several cases:
[1]. Faulty clients
[2]. Natural delays from different source
[3]. Bootstrap
[4]. Failure recovery

Jay's alternative proposal solves [1], perhaps solve [2] as well if we are
able to set a reasonable max.append.delay. But it does not seem address [3]
and [4]. I actually doubt if [3] and [4] are solvable because it looks the
CreateTime gap is unavoidable in those two cases.

Thanks,

Jiangjie (Becket) Qin


On Tue, Oct 13, 2015 at 3:23 PM, Guozhang Wang  wrote:

> Just to complete Jay's option, here is my understanding:
>
> 1. For log retention: if we want to remove data before time t, we look into
> the index file of each segment and find the largest timestamp t' < t, find
> the corresponding timestamp and start scanning to the end of the segment,
> if there is no entry with timestamp >= t, we can delete this segment; if a
> segment's index smallest timestamp is larger than t, we can skip that
> segment.
>
> 2. For log rolling: if we want to start a new segment after time t, we look
> into the active segment's index file, if the largest timestamp is already >
> t, we can roll a new segment immediately; if it is < t, we read its
> corresponding offset and start scanning to the end of the segment, if we
> find a record whose timestamp > t, we can roll a new segment.
>
> For log rol

Build failed in Jenkins: kafka-trunk-jdk8 #27

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[cshapi] KAFKA-2632: move fetchable check ahead in handleFetchResponse

[cshapi] MINOR: ignore wakeups when committing offsets on consumer close

--
[...truncated 4191 lines...]

org.apache.kafka.copycat.file.FileStreamSourceTaskTest > testNormalLifecycle 
PASSED

org.apache.kafka.copycat.file.FileStreamSourceTaskTest > testMissingTopic PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > testSourceTasks 
PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > 
testSourceTasksStdin PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > testTaskClass 
PASSED

org.apache.kafka.copycat.file.FileStreamSourceConnectorTest > 
testMultipleSourcesInvalid PASSED
:copycat:json:checkstyleMain
:copycat:json:compileTestJavawarning: [options] bootstrap class path not set in 
conjunction with -source 1.7
Note: 

 uses unchecked or unsafe operations.
Note: Recompile with -Xlint:unchecked for details.
1 warning

:copycat:json:processTestResources UP-TO-DATE
:copycat:json:testClasses
:copycat:json:checkstyleTest
:copycat:json:test

org.apache.kafka.copycat.json.JsonConverterTest > longToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCacheSchemaToJsonConversion PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
nullSchemaAndMapNonStringKeysToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > floatToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > booleanToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndMapToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > stringToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timestampToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCopycatSchemaMetadataTranslation PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timestampToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > decimalToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToCopycatStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToJsonNonStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > longToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mismatchSchemaJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testCacheSchemaToCopycatConversion PASSED

org.apache.kafka.copycat.json.JsonConverterTest > 
testJsonSchemaMetadataTranslation PASSED

org.apache.kafka.copycat.json.JsonConverterTest > bytesToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > shortToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > intToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > structToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > stringToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndArrayToJson 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > byteToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaPrimitiveToCopycat 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > byteToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > intToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > dateToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > noSchemaToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > noSchemaToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullSchemaAndPrimitiveToJson 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToJsonStringKeys PASSED

org.apache.kafka.copycat.json.JsonConverterTest > arrayToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > nullToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timeToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > structToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > shortToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > dateToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > doubleToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > timeToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > floatToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > decimalToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > arrayToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > booleanToJson PASSED

org.apache.kafka.copycat.json.JsonConverterTest > mapToCopycatNonStringKeys 
PASSED

org.apache.kafka.copycat.json.JsonConverterTest > bytesToCopycat PASSED

org.apache.kafka.copycat.json.JsonConverterTest > doubleToCopycat PASSED
:copycat:runtime:checkstyleMain
:copycat:runtime:compileTestJavawarning: [options] bootstr

Build failed in Jenkins: kafka-trunk-jdk7 #685

2015-10-13 Thread Apache Jenkins Server
See 

Changes:

[cshapi] KAFKA-2632: move fetchable check ahead in handleFetchResponse

[cshapi] MINOR: ignore wakeups when committing offsets on consumer close

--
[...truncated 2769 lines...]

kafka.log.LogTest > testParseTopicPartitionNameForEmptyName PASSED

kafka.log.LogTest > testOpenDeletesObsoleteFiles PASSED

kafka.log.LogTest > testSizeBasedLogRoll PASSED

kafka.log.LogTest > testTimeBasedLogRollJitter PASSED

kafka.log.LogTest > testParseTopicPartitionName PASSED

kafka.log.LogTest > testTruncateTo PASSED

kafka.log.LogTest > testCleanShutdownFile PASSED

kafka.log.OffsetIndexTest > lookupExtremeCases PASSED

kafka.log.OffsetIndexTest > appendTooMany PASSED

kafka.log.OffsetIndexTest > randomLookupTest PASSED

kafka.log.OffsetIndexTest > testReopen PASSED

kafka.log.OffsetIndexTest > appendOutOfOrder PASSED

kafka.log.OffsetIndexTest > truncate PASSED

kafka.log.LogSegmentTest > testRecoveryWithCorruptMessage PASSED

kafka.log.LogSegmentTest > testRecoveryFixesCorruptIndex PASSED

kafka.log.LogSegmentTest > testReadFromGap PASSED

kafka.log.LogSegmentTest > testTruncate PASSED

kafka.log.LogSegmentTest > testReadBeforeFirstOffset PASSED

kafka.log.LogSegmentTest > testCreateWithInitFileSizeAppendMessage PASSED

kafka.log.LogSegmentTest > testChangeFileSuffixes PASSED

kafka.log.LogSegmentTest > testMaxOffset PASSED

kafka.log.LogSegmentTest > testNextOffsetCalculation PASSED

kafka.log.LogSegmentTest > testReadOnEmptySegment PASSED

kafka.log.LogSegmentTest > testReadAfterLast PASSED

kafka.log.LogSegmentTest > testCreateWithInitFileSizeClearShutdown PASSED

kafka.log.LogSegmentTest > testTruncateFull PASSED

kafka.log.CleanerTest > testBuildOffsetMap PASSED

kafka.log.CleanerTest > testSegmentGrouping PASSED

kafka.log.CleanerTest > testCleanSegmentsWithAbort PASSED

kafka.log.CleanerTest > testSegmentGroupingWithSparseOffsets PASSED

kafka.log.CleanerTest > testRecoveryAfterCrash PASSED

kafka.log.CleanerTest > testCleaningWithDeletes PASSED

kafka.log.CleanerTest > testCleanSegments PASSED

kafka.log.CleanerTest > testCleaningWithUnkeyedMessages PASSED

kafka.log.OffsetMapTest > testClear PASSED

kafka.log.OffsetMapTest > testBasicValidation PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testHeartbeatWrongCoordinator PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testHeartbeatIllegalGeneration PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testGenerationIdIncrementsOnRebalance PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > testHeartbeatUnknownGroup 
PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testHeartbeatDuringRebalanceCausesRebalanceInProgress PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupSessionTimeoutTooLarge PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupSessionTimeoutTooSmall PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testCommitOffsetWithDefaultGeneration PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupWrongCoordinator PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupUnknownConsumerExistingGroup PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupUnknownPartitionAssignmentStrategy PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testCommitOffsetFromUnknownGroup PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupInconsistentPartitionAssignmentStrategy PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testJoinGroupUnknownConsumerNewGroup PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > testValidJoinGroup PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > 
testHeartbeatUnknownConsumerExistingGroup PASSED

kafka.coordinator.ConsumerCoordinatorResponseTest > testValidHeartbeat PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testDeadToRebalancingIllegalTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testPreparingRebalanceToRebalancingTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testRebalancingToDeadIllegalTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testPreparingRebalanceToStableIllegalTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testRebalancingToPreparingRebalanceTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > testCannotRebalanceWhenDead PASSED

kafka.coordinator.ConsumerGroupMetadataTest > testStableToDeadIllegalTransition 
PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testCannotRebalanceWhenPreparingRebalance PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testDeadToPreparingRebalanceIllegalTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > 
testStableToRebalancingIllegalTransition PASSED

kafka.coordinator.ConsumerGroupMetadataTest > testDeadToDeadIllegalTransition 
PASSED

kafka

[jira] [Commented] (KAFKA-2120) Add a request timeout to NetworkClient

2015-10-13 Thread Guozhang Wang (JIRA)

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

Guozhang Wang commented on KAFKA-2120:
--

[~mgharat] Your commit seems contains some debugging code in KafkaProducer like

{code}
remainingTime = checkMaybeGetRemainingTime(startTime);
{code}

Could you remove them?

> Add a request timeout to NetworkClient
> --
>
> Key: KAFKA-2120
> URL: https://issues.apache.org/jira/browse/KAFKA-2120
> Project: Kafka
>  Issue Type: New Feature
>Reporter: Jiangjie Qin
>Assignee: Mayuresh Gharat
>Priority: Blocker
> Fix For: 0.9.0.0
>
> Attachments: KAFKA-2120.patch, KAFKA-2120_2015-07-27_15:31:19.patch, 
> KAFKA-2120_2015-07-29_15:57:02.patch, KAFKA-2120_2015-08-10_19:55:18.patch, 
> KAFKA-2120_2015-08-12_10:59:09.patch, KAFKA-2120_2015-09-03_15:12:02.patch, 
> KAFKA-2120_2015-09-04_17:49:01.patch, KAFKA-2120_2015-09-09_16:45:44.patch, 
> KAFKA-2120_2015-09-09_18:56:18.patch, KAFKA-2120_2015-09-10_21:38:55.patch, 
> KAFKA-2120_2015-09-11_14:54:15.patch, KAFKA-2120_2015-09-15_18:57:20.patch, 
> KAFKA-2120_2015-09-18_19:27:48.patch, KAFKA-2120_2015-09-28_16:13:02.patch
>
>
> Currently NetworkClient does not have a timeout setting for requests. So if 
> no response is received for a request due to reasons such as broker is down, 
> the request will never be completed.
> Request timeout will also be used as implicit timeout for some methods such 
> as KafkaProducer.flush() and kafkaProducer.close().
> KIP-19 is created for this public interface change.
> https://cwiki.apache.org/confluence/display/KAFKA/KIP-19+-+Add+a+request+timeout+to+NetworkClient



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


Re: KIP-28 does not allow Processor to specify partition of output message

2015-10-13 Thread Guozhang Wang
I see your point. Yeah I think it is a good way to add a Partitioner into
addSink(...) but the Partitioner interface in producer is a bit overkill:

"partition(String topic, Object key, byte[] keyBytes, Object value, byte[]
valueBytes, Cluster cluster)"

whereas for us we only want to partition on (K key, V value).

Perhaps we should add a new Partitioner interface in Kafka Streams?

Guozhang

On Tue, Oct 13, 2015 at 6:38 PM, Randall Hauch  wrote:

> This overrides the partitioning logic for all topics, right? That means I
> have to explicitly call the default partitioning logic for all topics
> except those that my Producer forwards. I’m guess the best way to do by
> extending org.apache.kafka.clients.producer.DefaultProducer. Of course,
> with multiple sinks in my topology, I have to put all of the partitioning
> logic inside a single class.
>
> What would you think about adding an overloaded TopologyBuilder.addSink(…)
> method that takes a Partitioner (or better yet a smaller functional
> interface). The resulting SinkProcessor could use that Partitioner instance
> to set the partition number? That’d be super convenient for users, would
> keep the logic where it belongs (where the topology defines the sinks), and
> best of all the implementations won't have to worry about any other topics,
> such as those used by stores, metrics, or other sinks.
>
> Best regards,
>
> Randall
>
>
> On October 13, 2015 at 8:09:41 PM, Guozhang Wang (wangg...@gmail.com)
> wrote:
>
> Hi Randall,
>
> You can try to set the partitioner class as
> ProducerConfig.PARTITIONER_CLASS_CONFIG in the StreamsConfig, its
> interface
> can be found in
>
> org.apache.kafka.clients.producer.Partitioner
>
> Let me know if it works for you.
>
> Guozhang
>
> On Tue, Oct 13, 2015 at 10:59 AM, Randall Hauch  wrote:
>
> > The new streams API added with KIP-28 is great. I’ve been using it on a
> > prototype for a few weeks, and I’m looking forward to it being included
> in
> > 0.9.0. However, at the moment, a Processor implementation is not able to
> > specify the partition number when it outputs messages.
> >
> > I’d be happy to log a JIRA and create a PR to add it to the API, but
> > without knowing all of the history I’m wondering if leaving it out of
> the
> > API was intentional.
> >
> > Thoughts?
> >
> > Best regards,
> >
> > Randall Hauch
> >
>
>
>
> --
> -- Guozhang
>
>


-- 
-- Guozhang


[jira] [Commented] (KAFKA-2017) Persist Coordinator State for Coordinator Failover

2015-10-13 Thread Guozhang Wang (JIRA)

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

Guozhang Wang commented on KAFKA-2017:
--

I agree, and I think that was originally proposed by [~onurkaraman]. Onur do 
you want to chime in here?

> Persist Coordinator State for Coordinator Failover
> --
>
> Key: KAFKA-2017
> URL: https://issues.apache.org/jira/browse/KAFKA-2017
> Project: Kafka
>  Issue Type: Sub-task
>  Components: consumer
>Affects Versions: 0.9.0.0
>Reporter: Onur Karaman
>Assignee: Guozhang Wang
> Fix For: 0.9.0.0
>
> Attachments: KAFKA-2017.patch, KAFKA-2017_2015-05-20_09:13:39.patch, 
> KAFKA-2017_2015-05-21_19:02:47.patch
>
>
> When a coordinator fails, the group membership protocol tries to failover to 
> a new coordinator without forcing all the consumers rejoin their groups. This 
> is possible if the coordinator persists its state so that the state can be 
> transferred during coordinator failover. This state consists of most of the 
> information in GroupRegistry and ConsumerRegistry.



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


  1   2   >