diff --git a/ksql-engine/src/main/java/io/confluent/ksql/physical/PhysicalPlanBuilder.java b/ksql-engine/src/main/java/io/confluent/ksql/physical/PhysicalPlanBuilder.java index a4196c137fc7..9f0e8cc2c269 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/physical/PhysicalPlanBuilder.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/physical/PhysicalPlanBuilder.java @@ -201,7 +201,7 @@ private QueryMetadata buildPlanForStructuredOutputNode(String sqlExpression, fin .KSTREAM, applicationId, kafkaTopicClient, - ksqlConfig, outputNode.getSchema(), + outputNode.getSchema(), sinkDataSource.getKsqlTopic(), topologyDescription.toString()); } diff --git a/ksql-engine/src/main/java/io/confluent/ksql/util/PersistentQueryMetadata.java b/ksql-engine/src/main/java/io/confluent/ksql/util/PersistentQueryMetadata.java index d8928ade7426..4ade596f1f02 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/util/PersistentQueryMetadata.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/util/PersistentQueryMetadata.java @@ -41,12 +41,11 @@ public PersistentQueryMetadata(final String statementString, final DataSource.DataSourceType dataSourceType, final String queryApplicationId, final KafkaTopicClient kafkaTopicClient, - final KsqlConfig ksqlConfig, final Schema resultSchema, final KsqlTopic resultTopic, final String topology) { super(statementString, kafkaStreams, outputNode, executionPlan, dataSourceType, - queryApplicationId, kafkaTopicClient, ksqlConfig, topology); + queryApplicationId, kafkaTopicClient, topology); this.id = id; this.resultSchema = resultSchema; this.resultTopic = resultTopic; diff --git a/ksql-engine/src/main/java/io/confluent/ksql/util/QueryMetadata.java b/ksql-engine/src/main/java/io/confluent/ksql/util/QueryMetadata.java index bdfec255ec42..95543c9d4ce9 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/util/QueryMetadata.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/util/QueryMetadata.java @@ -35,7 +35,6 @@ public class QueryMetadata { private final DataSource.DataSourceType dataSourceType; private final String queryApplicationId; private final KafkaTopicClient kafkaTopicClient; - private final KsqlConfig ksqlConfig; private final String topoplogy; @@ -46,7 +45,6 @@ public QueryMetadata(final String statementString, final DataSource.DataSourceType dataSourceType, final String queryApplicationId, final KafkaTopicClient kafkaTopicClient, - final KsqlConfig ksqlConfig, String topoplogy) { this.statementString = statementString; this.kafkaStreams = kafkaStreams; @@ -55,7 +53,6 @@ public QueryMetadata(final String statementString, this.dataSourceType = dataSourceType; this.queryApplicationId = queryApplicationId; this.kafkaTopicClient = kafkaTopicClient; - this.ksqlConfig = ksqlConfig; this.topoplogy = topoplogy; } diff --git a/ksql-engine/src/main/java/io/confluent/ksql/util/QueuedQueryMetadata.java b/ksql-engine/src/main/java/io/confluent/ksql/util/QueuedQueryMetadata.java index 6c5348503e3a..ca452dc9cef4 100644 --- a/ksql-engine/src/main/java/io/confluent/ksql/util/QueuedQueryMetadata.java +++ b/ksql-engine/src/main/java/io/confluent/ksql/util/QueuedQueryMetadata.java @@ -42,7 +42,7 @@ public QueuedQueryMetadata( final KsqlConfig ksqlConfig ) { super(statementString, kafkaStreams, outputNode, executionPlan, dataSourceType, - queryApplicationId, kafkaTopicClient, ksqlConfig, "not-applicable-for-queued"); + queryApplicationId, kafkaTopicClient, "not-applicable-for-queued"); this.rowQueue = rowQueue; } diff --git a/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTest.java b/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTest.java index 4da837cbf9c4..66fa9631398c 100644 --- a/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTest.java +++ b/ksql-engine/src/test/java/io/confluent/ksql/KsqlContextTest.java @@ -79,8 +79,7 @@ private List getQueryMetadata(QueryId queryid, DataSource.DataSou type, "KSQL_query_" + queryid, null, - null, - null, + null, null, "topology"); diff --git a/ksql-engine/src/test/java/io/confluent/ksql/util/AvroUtilTest.java b/ksql-engine/src/test/java/io/confluent/ksql/util/AvroUtilTest.java index baab547a95bd..a39edf0cf7ee 100644 --- a/ksql-engine/src/test/java/io/confluent/ksql/util/AvroUtilTest.java +++ b/ksql-engine/src/test/java/io/confluent/ksql/util/AvroUtilTest.java @@ -23,7 +23,6 @@ import java.io.IOException; import java.util.Arrays; -import java.util.Collections; import java.util.HashMap; import java.util.List; @@ -121,8 +120,7 @@ public void shouldValidatePersistentQueryResultCorrectly() DataSource.DataSourceType.KSTREAM, "", mock(KafkaTopicClient.class), - new KsqlConfig(Collections.EMPTY_MAP), - resultSchema, + resultSchema, resultTopic, null); org.apache.avro.Schema.Parser parser = new org.apache.avro.Schema.Parser(); @@ -148,8 +146,7 @@ public void shouldFailForInvalidResultAvroSchema() DataSource.DataSourceType.KSTREAM, "", mock(KafkaTopicClient.class), - new KsqlConfig(Collections.EMPTY_MAP), - resultSchema, + resultSchema, resultTopic, null); expect(schemaRegistryClient.testCompatibility(anyString(), anyObject())).andReturn(false);