chia7712 commented on code in PR #20361:
URL: https://github.com/apache/kafka/pull/20361#discussion_r2292111738


##########
tests/kafkatest/services/performance/producer_performance.py:
##########
@@ -91,7 +91,7 @@ def start_cmd(self, node):
 
         cmd += " export KAFKA_LOG4J_OPTS=\"%s%s\"; " % 
(get_log4j_config_param(node), get_log4j_config_for_tools(node))
         cmd += "KAFKA_OPTS=%(kafka_opts)s 
KAFKA_HEAP_OPTS=\"-XX:+HeapDumpOnOutOfMemoryError\" %(kafka_run_class)s 
org.apache.kafka.tools.ProducerPerformance " \
-              "--topic %(topic)s --num-records %(num_records)d --record-size 
%(record_size)d --throughput %(throughput)d --producer-props 
bootstrap.servers=%(bootstrap_servers)s client.id=%(client_id)s 
%(metrics_props)s" % args
+              "--topic %(topic)s --num-records %(num_records)d --record-size 
%(record_size)d --throughput %(throughput)d --command-property 
bootstrap.servers=%(bootstrap_servers)s client.id=%(client_id)s 
%(metrics_props)s" % args

Review Comment:
   I assumed this change would cause an issue when running 
`ProducerPerformanceService` on an older kafka. However, it seems that 
`ProducerPerformanceService` always uses the tool jar from the development 
branch 



##########
tools/src/main/java/org/apache/kafka/tools/ProducerPerformance.java:
##########
@@ -230,40 +230,50 @@ static List<byte[]> readPayloadFile(String 
payloadFilePath, String payloadDelimi
     /** Get the command-line argument parser. */
     static ArgumentParser argParser() {
         ArgumentParser parser = ArgumentParsers
-                .newArgumentParser("producer-performance")
+                .newArgumentParser("kafka-producer-perf-test")
                 .defaultHelp(true)
                 .description("This tool is used to verify the producer 
performance. To enable transactions, " +
-                        "you can specify a transaction id or set a transaction 
duration using --transaction-duration-ms. " +
-                        "There are three ways to specify the transaction id: 
set transaction.id=<id> via --producer-props, " +
-                        "set transaction.id=<id> in the config file via 
--producer.config, or use --transaction-id <id>.");
+                        "you can specify a transactional id or set a 
transaction duration using --transaction-duration-ms. " +
+                        "There are three ways to specify the transactional id: 
set transactional.id=<id> via --command-property, " +
+                        "set transactional.id=<id> in the config file via 
--command-config, or use --transactional-id <id>.");
+
+        parser.addArgument("--bootstrap-server")

Review Comment:
   Should those configuration changes be documented in `upgrade.html`?



##########
tools/src/main/java/org/apache/kafka/tools/ProducerPerformance.java:
##########
@@ -540,43 +578,68 @@ static final class ConfigPostProcessor {
         final Long transactionDurationMs;
         final boolean transactionsEnabled;
         final List<byte[]> payloadByteList;
+        final long reportingInterval;
 
         public ConfigPostProcessor(ArgumentParser parser, String[] args) 
throws IOException, ArgumentParserException {
             Namespace namespace = parser.parseArgs(args);
+            this.bootstrapServers = namespace.getString("bootstrapServers");
             this.topicName = namespace.getString("topic");
             this.numRecords = namespace.getLong("numRecords");
             this.warmupRecords = Math.max(namespace.getLong("warmupRecords"), 
0);
             this.recordSize = namespace.getInt("recordSize");
             this.throughput = namespace.getDouble("throughput");
             this.payloadMonotonic = namespace.getBoolean("payloadMonotonic");
             this.shouldPrintMetrics = namespace.getBoolean("printMetrics");
+            this.reportingInterval = namespace.getLong("reportingInterval");
 
             List<String> producerConfigs = namespace.getList("producerConfig");
             String producerConfigFile = 
namespace.getString("producerConfigFile");
+            List<String> commandProperties = 
namespace.getList("commandProperties");
+            String commandConfigFile = 
namespace.getString("commandConfigFile");
             String payloadFilePath = namespace.getString("payloadFile");
             Long transactionDurationMsArg = 
namespace.getLong("transactionDurationMs");
             String transactionIdArg = namespace.getString("transactionalId");
             if (numRecords <= 0) {
-                throw new ArgumentParserException("--num-records should be 
greater than zero", parser);
+                throw new ArgumentParserException("--num-records should be 
greater than zero.", parser);
             }
             if (warmupRecords >= numRecords) {
                 throw new ArgumentParserException("The value for 
--warmup-records must be strictly fewer than the number of records in the test, 
--num-records.", parser);
             }
             if (recordSize != null && recordSize <= 0) {
-                throw new ArgumentParserException("--record-size should be 
greater than zero", parser);
+                throw new ArgumentParserException("--record-size should be 
greater than zero.", parser);
+            }
+            if (bootstrapServers == null && commandProperties == null && 
producerConfigs == null && producerConfigFile == null && commandConfigFile == 
null) {
+                throw new ArgumentParserException("At least one of 
--bootstrap-server, --command-property, --producer-props, --producer.config or 
--command-config must be specified.", parser);
+            }
+            if (commandProperties != null && producerConfigs != null) {
+                throw new ArgumentParserException("--command-property and 
--producer-props cannot be specified together.", parser);
             }
-            if (producerConfigs == null && producerConfigFile == null) {
-                throw new ArgumentParserException("Either --producer-props or 
--producer.config must be specified.", parser);
+            if (commandConfigFile != null && producerConfigFile != null) {

Review Comment:
   ditto



##########
tools/src/main/java/org/apache/kafka/tools/ProducerPerformance.java:
##########
@@ -540,43 +578,68 @@ static final class ConfigPostProcessor {
         final Long transactionDurationMs;
         final boolean transactionsEnabled;
         final List<byte[]> payloadByteList;
+        final long reportingInterval;
 
         public ConfigPostProcessor(ArgumentParser parser, String[] args) 
throws IOException, ArgumentParserException {
             Namespace namespace = parser.parseArgs(args);
+            this.bootstrapServers = namespace.getString("bootstrapServers");
             this.topicName = namespace.getString("topic");
             this.numRecords = namespace.getLong("numRecords");
             this.warmupRecords = Math.max(namespace.getLong("warmupRecords"), 
0);
             this.recordSize = namespace.getInt("recordSize");
             this.throughput = namespace.getDouble("throughput");
             this.payloadMonotonic = namespace.getBoolean("payloadMonotonic");
             this.shouldPrintMetrics = namespace.getBoolean("printMetrics");
+            this.reportingInterval = namespace.getLong("reportingInterval");
 
             List<String> producerConfigs = namespace.getList("producerConfig");
             String producerConfigFile = 
namespace.getString("producerConfigFile");
+            List<String> commandProperties = 
namespace.getList("commandProperties");
+            String commandConfigFile = 
namespace.getString("commandConfigFile");
             String payloadFilePath = namespace.getString("payloadFile");
             Long transactionDurationMsArg = 
namespace.getLong("transactionDurationMs");
             String transactionIdArg = namespace.getString("transactionalId");
             if (numRecords <= 0) {
-                throw new ArgumentParserException("--num-records should be 
greater than zero", parser);
+                throw new ArgumentParserException("--num-records should be 
greater than zero.", parser);
             }
             if (warmupRecords >= numRecords) {
                 throw new ArgumentParserException("The value for 
--warmup-records must be strictly fewer than the number of records in the test, 
--num-records.", parser);
             }
             if (recordSize != null && recordSize <= 0) {
-                throw new ArgumentParserException("--record-size should be 
greater than zero", parser);
+                throw new ArgumentParserException("--record-size should be 
greater than zero.", parser);
+            }
+            if (bootstrapServers == null && commandProperties == null && 
producerConfigs == null && producerConfigFile == null && commandConfigFile == 
null) {
+                throw new ArgumentParserException("At least one of 
--bootstrap-server, --command-property, --producer-props, --producer.config or 
--command-config must be specified.", parser);
+            }
+            if (commandProperties != null && producerConfigs != null) {

Review Comment:
   it seems this case does not have a unit test



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: jira-unsubscr...@kafka.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org

Reply via email to