mirror of https://github.com/apache/kafka.git
KAFKA-16448: Unify class cast exception handling for both key and value (#16736)
Part of KIP-1033. Minor code cleanup. Reviewers: Matthias J. Sax <matthias@confluent.io>
This commit is contained in:
parent
8f2679bebf
commit
0eb9ac2bd0
|
@ -53,9 +53,11 @@ import org.apache.kafka.streams.processor.internals.metrics.TopicMetrics;
|
||||||
|
|
||||||
import org.slf4j.Logger;
|
import org.slf4j.Logger;
|
||||||
|
|
||||||
|
import java.text.MessageFormat;
|
||||||
import java.util.Collections;
|
import java.util.Collections;
|
||||||
import java.util.HashMap;
|
import java.util.HashMap;
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
|
import java.util.Locale;
|
||||||
import java.util.Map;
|
import java.util.Map;
|
||||||
import java.util.Optional;
|
import java.util.Optional;
|
||||||
import java.util.Set;
|
import java.util.Set;
|
||||||
|
@ -200,7 +202,8 @@ public class RecordCollectorImpl implements RecordCollector {
|
||||||
try {
|
try {
|
||||||
keyBytes = keySerializer.serialize(topic, headers, key);
|
keyBytes = keySerializer.serialize(topic, headers, key);
|
||||||
} catch (final ClassCastException exception) {
|
} catch (final ClassCastException exception) {
|
||||||
throw createStreamsExceptionForKeyClassCastException(
|
throw createStreamsExceptionForClassCastException(
|
||||||
|
ProductionExceptionHandler.SerializationExceptionOrigin.KEY,
|
||||||
topic,
|
topic,
|
||||||
key,
|
key,
|
||||||
keySerializer,
|
keySerializer,
|
||||||
|
@ -223,7 +226,8 @@ public class RecordCollectorImpl implements RecordCollector {
|
||||||
try {
|
try {
|
||||||
valBytes = valueSerializer.serialize(topic, headers, value);
|
valBytes = valueSerializer.serialize(topic, headers, value);
|
||||||
} catch (final ClassCastException exception) {
|
} catch (final ClassCastException exception) {
|
||||||
throw createStreamsExceptionForValueClassCastException(
|
throw createStreamsExceptionForClassCastException(
|
||||||
|
ProductionExceptionHandler.SerializationExceptionOrigin.VALUE,
|
||||||
topic,
|
topic,
|
||||||
value,
|
value,
|
||||||
valueSerializer,
|
valueSerializer,
|
||||||
|
@ -335,39 +339,27 @@ public class RecordCollectorImpl implements RecordCollector {
|
||||||
|
|
||||||
droppedRecordsSensor.record();
|
droppedRecordsSensor.record();
|
||||||
}
|
}
|
||||||
private <K> StreamsException createStreamsExceptionForKeyClassCastException(final String topic,
|
|
||||||
final K key,
|
|
||||||
final Serializer<K> keySerializer,
|
|
||||||
final ClassCastException exception) {
|
|
||||||
final String keyClass = key == null ? "unknown because key is null" : key.getClass().getName();
|
|
||||||
return new StreamsException(
|
|
||||||
String.format(
|
|
||||||
"ClassCastException while producing data to topic %s. " +
|
|
||||||
"The key serializer %s is not compatible to the actual key type: %s. " +
|
|
||||||
"Change the default key serde in StreamConfig or provide the correct key serde via method parameters " +
|
|
||||||
"(for example if using the DSL, `#to(String topic, Produced<K, V> produced)` with " +
|
|
||||||
"`Produced.keySerde(WindowedSerdes.timeWindowedSerdeFrom(String.class))`).",
|
|
||||||
topic,
|
|
||||||
keySerializer.getClass().getName(),
|
|
||||||
keyClass),
|
|
||||||
exception);
|
|
||||||
}
|
|
||||||
|
|
||||||
private <V> StreamsException createStreamsExceptionForValueClassCastException(final String topic,
|
private <KV> StreamsException createStreamsExceptionForClassCastException(final ProductionExceptionHandler.SerializationExceptionOrigin origin,
|
||||||
final V value,
|
final String topic,
|
||||||
final Serializer<V> valueSerializer,
|
final KV keyOrValue,
|
||||||
|
final Serializer<KV> keyOrValueSerializer,
|
||||||
final ClassCastException exception) {
|
final ClassCastException exception) {
|
||||||
final String valueClass = value == null ? "unknown because value is null" : value.getClass().getName();
|
final String keyOrValueClass = keyOrValue == null
|
||||||
|
? String.format("unknown because %s is null", origin.toString().toLowerCase(Locale.ROOT)) : keyOrValue.getClass().getName();
|
||||||
|
|
||||||
return new StreamsException(
|
return new StreamsException(
|
||||||
|
MessageFormat.format(
|
||||||
String.format(
|
String.format(
|
||||||
"ClassCastException while producing data to topic %s. " +
|
"ClassCastException while producing data to topic %s. " +
|
||||||
"The value serializer %s is not compatible to the actual value type: %s. " +
|
"The {0} serializer %s is not compatible to the actual {0} type: %s. " +
|
||||||
"Change the default value serde in StreamConfig or provide the correct value serde via method parameters " +
|
"Change the default {0} serde in StreamConfig or provide the correct {0} serde via method parameters " +
|
||||||
"(for example if using the DSL, `#to(String topic, Produced<K, V> produced)` with " +
|
"(for example if using the DSL, `#to(String topic, Produced<K, V> produced)` with " +
|
||||||
"`Produced.valueSerde(WindowedSerdes.timeWindowedSerdeFrom(String.class))`).",
|
"`Produced.{0}Serde(WindowedSerdes.timeWindowedSerdeFrom(String.class))`).",
|
||||||
topic,
|
topic,
|
||||||
valueSerializer.getClass().getName(),
|
keyOrValueSerializer.getClass().getName(),
|
||||||
valueClass),
|
keyOrValueClass),
|
||||||
|
origin.toString().toLowerCase(Locale.ROOT)),
|
||||||
exception);
|
exception);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
Loading…
Reference in New Issue