You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@kafka.apache.org by gw...@apache.org on 2015/11/07 02:55:42 UTC

kafka git commit: KAFKA-2766: use new producer by default in tooling

Repository: kafka
Updated Branches:
  refs/heads/trunk d297b3af2 -> 15524d097


KAFKA-2766: use new producer by default in tooling

Also update the API docs for new consumer.

Author: Guozhang Wang <wa...@gmail.com>

Reviewers: Gwen Shapira

Closes #448 from guozhangwang/K2766


Project: http://git-wip-us.apache.org/repos/asf/kafka/repo
Commit: http://git-wip-us.apache.org/repos/asf/kafka/commit/15524d09
Tree: http://git-wip-us.apache.org/repos/asf/kafka/tree/15524d09
Diff: http://git-wip-us.apache.org/repos/asf/kafka/diff/15524d09

Branch: refs/heads/trunk
Commit: 15524d0970868d2dd9bc16b9d5de4519fc469e11
Parents: d297b3a
Author: Guozhang Wang <wa...@gmail.com>
Authored: Fri Nov 6 17:55:32 2015 -0800
Committer: Gwen Shapira <cs...@gmail.com>
Committed: Fri Nov 6 17:55:32 2015 -0800

----------------------------------------------------------------------
 .../main/scala/kafka/tools/ConsoleProducer.scala    | 10 +++++-----
 .../scala/kafka/tools/ProducerPerformance.scala     |  3 +++
 .../scala/kafka/tools/ConsoleProducerTest.scala     |  5 ++---
 docs/api.html                                       | 16 +++++++++++++++-
 docs/upgrade.html                                   |  2 ++
 5 files changed, 27 insertions(+), 9 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/kafka/blob/15524d09/core/src/main/scala/kafka/tools/ConsoleProducer.scala
----------------------------------------------------------------------
diff --git a/core/src/main/scala/kafka/tools/ConsoleProducer.scala b/core/src/main/scala/kafka/tools/ConsoleProducer.scala
index de50f65..ff1bd70 100644
--- a/core/src/main/scala/kafka/tools/ConsoleProducer.scala
+++ b/core/src/main/scala/kafka/tools/ConsoleProducer.scala
@@ -39,10 +39,10 @@ object ConsoleProducer {
         reader.init(System.in, getReaderProps(config))
 
         val producer =
-          if(config.useNewProducer) {
-            new NewShinyProducer(getNewProducerProps(config))
-          } else {
+          if(config.useOldProducer) {
             new OldProducer(getOldProducerProps(config))
+          } else {
+            new NewShinyProducer(getNewProducerProps(config))
           }
 
         Runtime.getRuntime.addShutdownHook(new Thread() {
@@ -239,7 +239,7 @@ object ConsoleProducer {
             .withRequiredArg
             .describedAs("producer_prop")
             .ofType(classOf[String])
-    val useNewProducerOpt = parser.accepts("new-producer", "Use the new producer implementation.")
+    val useOldProducerOpt = parser.accepts("old-producer", "Use the old producer implementation.")
 
     val options = parser.parse(args : _*)
     if(args.length == 0)
@@ -247,7 +247,7 @@ object ConsoleProducer {
     CommandLineUtils.checkRequiredArgs(parser, options, topicOpt, brokerListOpt)
 
     import scala.collection.JavaConversions._
-    val useNewProducer = options.has(useNewProducerOpt)
+    val useOldProducer = options.has(useOldProducerOpt)
     val topic = options.valueOf(topicOpt)
     val brokerList = options.valueOf(brokerListOpt)
     ToolsUtils.validatePortOrDie(parser,brokerList)

http://git-wip-us.apache.org/repos/asf/kafka/blob/15524d09/core/src/main/scala/kafka/tools/ProducerPerformance.scala
----------------------------------------------------------------------
diff --git a/core/src/main/scala/kafka/tools/ProducerPerformance.scala b/core/src/main/scala/kafka/tools/ProducerPerformance.scala
index e299f8b..7fab9d0 100644
--- a/core/src/main/scala/kafka/tools/ProducerPerformance.scala
+++ b/core/src/main/scala/kafka/tools/ProducerPerformance.scala
@@ -34,7 +34,10 @@ import org.apache.log4j.Logger
 
 /**
  * Load test for the producer
+ *
+ * @note This class will be replaced by org.apache.kafka.tools.ProducerPerformance after the old producer client is removed
  */
+@deprecated
 object ProducerPerformance extends Logging {
 
   def main(args: Array[String]) {

http://git-wip-us.apache.org/repos/asf/kafka/blob/15524d09/core/src/test/scala/kafka/tools/ConsoleProducerTest.scala
----------------------------------------------------------------------
diff --git a/core/src/test/scala/kafka/tools/ConsoleProducerTest.scala b/core/src/test/scala/kafka/tools/ConsoleProducerTest.scala
index 6d1f51c..7e211b7 100644
--- a/core/src/test/scala/kafka/tools/ConsoleProducerTest.scala
+++ b/core/src/test/scala/kafka/tools/ConsoleProducerTest.scala
@@ -18,9 +18,8 @@
 package kafka.tools
 
 import kafka.producer
-import kafka.tools.ConsoleProducer.{LineMessageReader, MessageReader,ProducerConfig}
-import org.apache.kafka.clients.producer.{KafkaProducer, ProducerConfig}
-import joptsimple.UnrecognizedOptionException
+import kafka.tools.ConsoleProducer.LineMessageReader
+import org.apache.kafka.clients.producer.KafkaProducer
 import org.junit.Assert
 import org.junit.Test
 

http://git-wip-us.apache.org/repos/asf/kafka/blob/15524d09/docs/api.html
----------------------------------------------------------------------
diff --git a/docs/api.html b/docs/api.html
index 2b8bad7..1787f06 100644
--- a/docs/api.html
+++ b/docs/api.html
@@ -19,7 +19,7 @@ We are in the process of rewritting the JVM clients for Kafka. As of 0.8.2 Kafka
 
 <h3><a id="producerapi">2.1 Producer API</a></h3>
 
-As of the 0.8.2 release we encourage all new development to use the new Java producer. This client is production tested and generally both faster and more fully featured than the previous Scala client. You can use this client by adding a dependency on the client jar using the following maven co-ordinates:
+As of the 0.8.2 release we encourage all new development to use the new Java producer. This client is production tested and generally both faster and more fully featured than the previous Scala client. You can use this client by adding a dependency on the client jar using the following example maven co-ordinates (you can change the version numbers with new releases):
 <pre>
 	&lt;dependency&gt;
 	    &lt;groupId&gt;org.apache.kafka&lt;/groupId&gt;
@@ -152,3 +152,17 @@ Providing a horizontally scalable solution for aggregating and loading data into
 <p>
 Usage information on the hadoop consumer can be found <a href="https://github.com/linkedin/camus/">here</a>.
 </p>
+
+
+<h3><a id="newconsumerapi">2.5 New Consumer API</a></h3>
+As of the 0.9.0 release we have added a replacement for our existing simple and high-level consumers. This client is considered beta quality. You can use this client by adding a dependency on the client jar using the following example maven co-ordinates (you can change the version numbers with new releases):
+<pre>
+	&lt;dependency&gt;
+	    &lt;groupId&gt;org.apache.kafka&lt;/groupId&gt;
+	    &lt;artifactId&gt;kafka-clients&lt;/artifactId&gt;
+	    &lt;version&gt;0.9.0.0&lt;/version&gt;
+	&lt;/dependency&gt;
+</pre>
+
+Examples showing how to use the producer are given in the
+<a href="http://kafka.apache.org/090/javadoc/index.html?org/apache/kafka/clients/producer/KafkaConsumer.html" title="Kafka 0.9.0 Javadoc">javadocs</a>.
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/kafka/blob/15524d09/docs/upgrade.html
----------------------------------------------------------------------
diff --git a/docs/upgrade.html b/docs/upgrade.html
index 69bcdc1..69ff20a 100644
--- a/docs/upgrade.html
+++ b/docs/upgrade.html
@@ -39,6 +39,7 @@ Note: Bumping the protocol version and restarting can be done any time after the
     <li> The default Kafka JVM performance options (KAFKA_JVM_PERFORMANCE_OPTS) have been changed in kafka-run-class.sh. </li>
     <li> The kafka-topics.sh script (kafka.admin.TopicCommand) now exits with non-zero exit code on failure. </li>
     <li> The kafka-topics.sh script (kafka.admin.TopicCommand) will now print a warning when topic names risk metric collisions due to the use of a '.' or '_' in the topic name, and error in the case of an actual collision. </li>
+    <li> The kafka-console-producer.sh script (kafka.tools.ConsoleProducer) will use the new producer instead of the old producer be default, and users have to specify 'old-producer' to use the old producer. </li>
     <li> By default all command line tools will print all logging messages to stderr instead of stout. </li>
 </ul>
 
@@ -47,6 +48,7 @@ Note: Bumping the protocol version and restarting can be done any time after the
 <ul>
     <li> Altering topic configuration from the kafka-topics.sh script (kafka.admin.TopicCommand) has been deprecated. Going forward, please use the kafka-configs.sh script (kafka.admin.ConfigCommand) for this functionality. </li>
     <li> The kafka-consumer-offset-checker.sh (kafka.tools.ConsumerOffsetChecker) has been deprecated. Going forward, please use kafka-consumer-groups.sh (kafka.admin.ConsumerGroupCommand) for this functionality. </li>
+    <li> The kafka.tools.ProducerPerformance class has been deprecated. Going forward, please use org.apache.kafka.tools.ProducerPerformance for this functionality (kafka-producer-perf-test.sh will also be changed to use the new class). </li>
 </ul>
 
 <h4>Upgrading from 0.8.1 to 0.8.2.0</h4>