Skip to content

Commit

Permalink
fix: INSERT/VALUES on a stream with SCHEMA_ID/SCHEMA_FULL_NAME fails (#…
Browse files Browse the repository at this point in the history
  • Loading branch information
spena committed Apr 22, 2022
1 parent 66c59a5 commit bfd910b
Show file tree
Hide file tree
Showing 15 changed files with 1,593 additions and 3 deletions.
Expand Up @@ -515,11 +515,13 @@ private static void throwOnMultiSchemaDefinitions(
}
}

// CHECKSTYLE_RULES.OFF: CyclomaticComplexity
private static CreateSource addSchemaFields(
final ConfiguredStatement<CreateSource> preparedStatement,
final Optional<SchemaAndId> keySchema,
final Optional<SchemaAndId> valueSchema
) {
// CHECKSTYLE_RULES.ON: CyclomaticComplexity
final TableElements elements = buildElements(preparedStatement, keySchema, valueSchema);

final CreateSource statement = preparedStatement.getStatement();
Expand All @@ -542,13 +544,20 @@ private static CreateSource addSchemaFields(
throwOnMultiSchemaDefinitions(valueSchema.get().rawSchema, valueFormat, false);
}

// Only populate key and value schema names when schema ids are explicitly provided
if (properties.getKeySchemaId().isPresent() && keySchema.isPresent()) {
// Only populate key and value schema names when schema ids or full schema names are
// explicitly provided

if (properties.getKeySchemaFullName().isPresent()) {
keySchemaName = properties.getKeySchemaFullName();
} else if (properties.getKeySchemaId().isPresent() && keySchema.isPresent()) {
keySchemaName = Optional.ofNullable(keySchema.get().rawSchema.name());
} else {
keySchemaName = Optional.empty();
}
if (properties.getValueSchemaId().isPresent() && valueSchema.isPresent()) {

if (properties.getValueSchemaFullName().isPresent()) {
valueSchemaName = properties.getValueSchemaFullName();
} else if (properties.getValueSchemaId().isPresent() && valueSchema.isPresent()) {
valueSchemaName = Optional.ofNullable(valueSchema.get().rawSchema.name());
} else {
valueSchemaName = Optional.empty();
Expand Down
@@ -0,0 +1,223 @@
{
"plan" : [ {
"@type" : "ksqlPlanV1",
"statementText" : "CREATE STREAM INPUT (K1 INTEGER KEY, C1 INTEGER) WITH (FORMAT='PROTOBUF', KAFKA_TOPIC='input', KEY_SCHEMA_FULL_NAME='KeySchema2', KEY_SCHEMA_ID=1, VALUE_SCHEMA_FULL_NAME='ValueSchema2', VALUE_SCHEMA_ID=2);",
"ddlCommand" : {
"@type" : "createStreamV1",
"sourceName" : "INPUT",
"schema" : "`K1` INTEGER KEY, `C1` INTEGER",
"topicName" : "input",
"formats" : {
"keyFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "KeySchema2",
"unwrapPrimitives" : "true",
"schemaId" : "1"
}
},
"valueFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "ValueSchema2",
"unwrapPrimitives" : "true",
"schemaId" : "2"
}
}
},
"orReplace" : false,
"isSource" : false
}
}, {
"@type" : "ksqlPlanV1",
"statementText" : "CREATE STREAM OUTPUT AS SELECT *\nFROM INPUT INPUT\nEMIT CHANGES",
"ddlCommand" : {
"@type" : "createStreamV1",
"sourceName" : "OUTPUT",
"schema" : "`K1` INTEGER KEY, `C1` INTEGER",
"topicName" : "OUTPUT",
"formats" : {
"keyFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "KeySchema2",
"unwrapPrimitives" : "true"
}
},
"valueFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "ValueSchema2",
"unwrapPrimitives" : "true"
}
}
},
"orReplace" : false,
"isSource" : false
},
"queryPlan" : {
"sources" : [ "INPUT" ],
"sink" : "OUTPUT",
"physicalPlan" : {
"@type" : "streamSinkV1",
"properties" : {
"queryContext" : "OUTPUT"
},
"source" : {
"@type" : "streamSelectV1",
"properties" : {
"queryContext" : "Project"
},
"source" : {
"@type" : "streamSourceV1",
"properties" : {
"queryContext" : "KsqlTopic/Source"
},
"topicName" : "input",
"formats" : {
"keyFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "KeySchema2",
"unwrapPrimitives" : "true",
"schemaId" : "1"
}
},
"valueFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "ValueSchema2",
"unwrapPrimitives" : "true",
"schemaId" : "2"
}
}
},
"sourceSchema" : "`K1` INTEGER KEY, `C1` INTEGER",
"pseudoColumnVersion" : 1
},
"keyColumnNames" : [ "K1" ],
"selectExpressions" : [ "C1 AS C1" ]
},
"formats" : {
"keyFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "KeySchema2",
"unwrapPrimitives" : "true"
}
},
"valueFormat" : {
"format" : "PROTOBUF",
"properties" : {
"fullSchemaName" : "ValueSchema2",
"unwrapPrimitives" : "true"
}
}
},
"topicName" : "OUTPUT"
},
"queryId" : "CSAS_OUTPUT_0"
}
} ],
"configs" : {
"ksql.extension.dir" : "ext",
"ksql.streams.cache.max.bytes.buffering" : "0",
"metric.reporters" : "",
"ksql.query.status.running.threshold.seconds" : "300",
"ksql.connect.basic.auth.credentials.reload" : "false",
"ksql.output.topic.name.prefix" : "",
"ksql.query.pull.stream.enabled" : "true",
"ksql.query.push.v2.interpreter.enabled" : "true",
"ksql.queryanonymizer.logs_enabled" : "true",
"ksql.variable.substitution.enable" : "true",
"ksql.streams.shutdown.timeout.ms" : "300000",
"ksql.query.pull.max.allowed.offset.lag" : "9223372036854775807",
"ksql.query.pull.max.qps" : "2147483647",
"ksql.access.validator.enable" : "auto",
"ksql.streams.bootstrap.servers" : "localhost:0",
"ksql.query.pull.metrics.enabled" : "true",
"ksql.metrics.extension" : null,
"ksql.query.push.v2.alos.enabled" : "true",
"ksql.query.push.v2.max.hourly.bandwidth.megabytes" : "2147483647",
"ksql.query.pull.range.scan.enabled" : "true",
"ksql.transient.query.cleanup.service.initial.delay.seconds" : "600",
"ksql.hidden.topics" : "_confluent.*,__confluent.*,_schemas,__consumer_offsets,__transaction_state,connect-configs,connect-offsets,connect-status,connect-statuses",
"ksql.lambdas.enabled" : "true",
"ksql.source.table.materialization.enabled" : "true",
"ksql.query.pull.max.hourly.bandwidth.megabytes" : "2147483647",
"ksql.sink.window.change.log.additional.retention" : "1000000",
"ksql.query.persistent.active.limit" : "2147483647",
"ksql.persistence.wrap.single.values" : null,
"ksql.query.transient.max.bytes.buffering.total" : "-1",
"ksql.connect.basic.auth.credentials.source" : "NONE",
"ksql.schema.registry.url" : "schema_registry.url:0",
"ksql.properties.overrides.denylist" : "",
"ksql.service.id" : "some.ksql.service.id",
"ksql.query.push.v2.max.catchup.consumers" : "5",
"ksql.query.push.v2.enabled" : "false",
"ksql.transient.query.cleanup.service.enable" : "true",
"ksql.query.push.v2.metrics.enabled" : "true",
"ksql.rowpartition.rowoffset.enabled" : "true",
"ksql.streams.commit.interval.ms" : "2000",
"ksql.query.pull.table.scan.enabled" : "true",
"ksql.streams.auto.commit.interval.ms" : "0",
"ksql.streams.topology.optimization" : "all",
"ksql.endpoint.migrate.query" : "true",
"ksql.query.push.v2.registry.installed" : "false",
"ksql.streams.num.stream.threads" : "4",
"ksql.metrics.tags.custom" : "",
"ksql.query.push.v2.catchup.consumer.msg.window" : "50",
"ksql.runtime.feature.shared.enabled" : "false",
"ksql.udf.collect.metrics" : "false",
"ksql.new.query.planner.enabled" : "false",
"ksql.connect.request.headers.plugin" : null,
"ksql.security.extension.class" : null,
"ksql.transient.prefix" : "transient_",
"ksql.headers.columns.enabled" : "true",
"ksql.streams.default.deserialization.exception.handler" : "io.confluent.ksql.errors.LogMetricAndContinueExceptionHandler",
"ksql.connect.request.timeout.ms" : "5000",
"ksql.query.pull.enable.standby.reads" : "false",
"ksql.persistence.default.format.key" : "KAFKA",
"ksql.query.persistent.max.bytes.buffering.total" : "-1",
"ksql.query.error.max.queue.size" : "10",
"ksql.query.cleanup.shutdown.timeout.ms" : "30000",
"ksql.internal.topic.min.insync.replicas" : "1",
"ksql.internal.topic.replicas" : "1",
"ksql.insert.into.values.enabled" : "true",
"ksql.queryanonymizer.cluster_namespace" : null,
"ksql.create.or.replace.enabled" : "true",
"ksql.shared.runtimes.count" : "2",
"ksql.cast.strings.preserve.nulls" : "true",
"ksql.authorization.cache.max.entries" : "10000",
"ksql.pull.queries.enable" : "true",
"ksql.transient.query.cleanup.service.period.seconds" : "600",
"ksql.suppress.enabled" : "false",
"ksql.readonly.topics" : "_confluent.*,__confluent.*,_schemas,__consumer_offsets,__transaction_state,connect-configs,connect-offsets,connect-status,connect-statuses",
"ksql.connect.basic.auth.credentials.file" : "",
"ksql.authorization.cache.expiry.time.secs" : "30",
"ksql.query.retry.backoff.initial.ms" : "15000",
"ksql.query.pull.max.concurrent.requests" : "2147483647",
"ksql.streams.auto.offset.reset" : "earliest",
"ksql.connect.url" : "http://localhost:8083",
"ksql.query.push.v2.new.latest.delay.ms" : "5000",
"ksql.query.push.v2.latest.reset.age.ms" : "30000",
"ksql.streams.default.production.exception.handler" : "io.confluent.ksql.errors.ProductionExceptionHandlerUtil$LogAndFailProductionExceptionHandler",
"ksql.query.pull.interpreter.enabled" : "true",
"ksql.query.pull.limit.clause.enabled" : "true",
"ksql.connect.error.handler" : null,
"ksql.query.pull.router.thread.pool.size" : "50",
"ksql.query.push.v2.continuation.tokens.enabled" : "false",
"ksql.query.retry.backoff.max.ms" : "900000",
"ksql.timestamp.throw.on.invalid" : "false",
"ksql.persistence.default.format.value" : null,
"ksql.udfs.enabled" : "true",
"ksql.udf.enable.security.manager" : "true",
"ksql.connect.worker.config" : "",
"ksql.nested.error.set.null" : "true",
"ksql.query.pull.thread.pool.size" : "50",
"ksql.persistent.prefix" : "query_",
"ksql.metastore.backup.location" : "",
"ksql.error.classifier.regex" : "",
"ksql.suppress.buffer.size.bytes" : "-1"
}
}

0 comments on commit bfd910b

Please sign in to comment.