mirror of https://github.com/apache/kafka.git
MINOR: Improve checks for CogroupedStreamAggregateBuilder (#9141)
Update `CogroupedStreamAggregateBuilder` to have individual builders depending on the windowed aggregation, or lack thereof. This replaced passing in all options into the builder, with all but the current type of aggregation set to null and then checking to see which value was not null. Reviewers: A. Sophie Blee-Goldman <sophie@confluent.io>, John Roesler <vvcephei@apache.org>
This commit is contained in:
parent
c8bccdd913
commit
3e6dcb14dd
|
@ -129,9 +129,6 @@ public class CogroupedKStreamImpl<K, VOut> extends AbstractStream<K, VOut> imple
|
|||
new TimestampedKeyValueStoreMaterializer<>(materializedInternal).materialize(),
|
||||
materializedInternal.keySerde(),
|
||||
materializedInternal.valueSerde(),
|
||||
materializedInternal.queryableStoreName(),
|
||||
null,
|
||||
null,
|
||||
null);
|
||||
materializedInternal.queryableStoreName());
|
||||
}
|
||||
}
|
|
@ -47,18 +47,93 @@ class CogroupedStreamAggregateBuilder<K, VOut> {
|
|||
CogroupedStreamAggregateBuilder(final InternalStreamsBuilder builder) {
|
||||
this.builder = builder;
|
||||
}
|
||||
<KR> KTable<KR, VOut> build(final Map<KGroupedStreamImpl<K, ?>, Aggregator<? super K, ? super Object, VOut>> groupPatterns,
|
||||
final Initializer<VOut> initializer,
|
||||
final NamedInternal named,
|
||||
final StoreBuilder<?> storeBuilder,
|
||||
final Serde<KR> keySerde,
|
||||
final Serde<VOut> valueSerde,
|
||||
final String queryableName) {
|
||||
processRepartitions(groupPatterns, storeBuilder);
|
||||
final Collection<StreamsGraphNode> processors = new ArrayList<>();
|
||||
boolean stateCreated = false;
|
||||
int counter = 0;
|
||||
for (final Entry<KGroupedStreamImpl<K, ?>, Aggregator<? super K, Object, VOut>> kGroupedStream : groupPatterns.entrySet()) {
|
||||
final StatefulProcessorNode<K, ?> statefulProcessorNode = getStatefulProcessorNode(
|
||||
named.suffixWithOrElseGet(
|
||||
"-cogroup-agg-" + counter++,
|
||||
builder,
|
||||
CogroupedKStreamImpl.AGGREGATE_NAME),
|
||||
stateCreated,
|
||||
storeBuilder,
|
||||
new KStreamAggregate<>(storeBuilder.name(), initializer, kGroupedStream.getValue()));
|
||||
stateCreated = true;
|
||||
processors.add(statefulProcessorNode);
|
||||
builder.addGraphNode(parentNodes.get(kGroupedStream.getKey()), statefulProcessorNode);
|
||||
}
|
||||
return createTable(processors, named, keySerde, valueSerde, queryableName);
|
||||
}
|
||||
|
||||
<KR, VIn, W extends Window> KTable<KR, VOut> build(final Map<KGroupedStreamImpl<K, ?>, Aggregator<? super K, ? super Object, VOut>> groupPatterns,
|
||||
<KR, W extends Window> KTable<KR, VOut> build(final Map<KGroupedStreamImpl<K, ?>, Aggregator<? super K, ? super Object, VOut>> groupPatterns,
|
||||
final Initializer<VOut> initializer,
|
||||
final NamedInternal named,
|
||||
final StoreBuilder<?> storeBuilder,
|
||||
final Serde<KR> keySerde,
|
||||
final Serde<VOut> valueSerde,
|
||||
final String queryableName,
|
||||
final Windows<W> windows) {
|
||||
processRepartitions(groupPatterns, storeBuilder);
|
||||
|
||||
final Collection<StreamsGraphNode> processors = new ArrayList<>();
|
||||
boolean stateCreated = false;
|
||||
int counter = 0;
|
||||
for (final Entry<KGroupedStreamImpl<K, ?>, Aggregator<? super K, Object, VOut>> kGroupedStream : groupPatterns.entrySet()) {
|
||||
final StatefulProcessorNode<K, ?> statefulProcessorNode = getStatefulProcessorNode(
|
||||
named.suffixWithOrElseGet(
|
||||
"-cogroup-agg-" + counter++,
|
||||
builder,
|
||||
CogroupedKStreamImpl.AGGREGATE_NAME),
|
||||
stateCreated,
|
||||
storeBuilder,
|
||||
new KStreamWindowAggregate<>(windows, storeBuilder.name(), initializer, kGroupedStream.getValue()));
|
||||
stateCreated = true;
|
||||
processors.add(statefulProcessorNode);
|
||||
builder.addGraphNode(parentNodes.get(kGroupedStream.getKey()), statefulProcessorNode);
|
||||
}
|
||||
return createTable(processors, named, keySerde, valueSerde, queryableName);
|
||||
}
|
||||
|
||||
<KR> KTable<KR, VOut> build(final Map<KGroupedStreamImpl<K, ?>, Aggregator<? super K, ? super Object, VOut>> groupPatterns,
|
||||
final Initializer<VOut> initializer,
|
||||
final NamedInternal named,
|
||||
final StoreBuilder<?> storeBuilder,
|
||||
final Serde<KR> keySerde,
|
||||
final Serde<VOut> valueSerde,
|
||||
final String queryableName,
|
||||
final Windows<W> windows,
|
||||
final SessionWindows sessionWindows,
|
||||
final Merger<? super K, VOut> sessionMerger) {
|
||||
processRepartitions(groupPatterns, storeBuilder);
|
||||
final Collection<StreamsGraphNode> processors = new ArrayList<>();
|
||||
boolean stateCreated = false;
|
||||
int counter = 0;
|
||||
for (final Entry<KGroupedStreamImpl<K, ?>, Aggregator<? super K, Object, VOut>> kGroupedStream : groupPatterns.entrySet()) {
|
||||
final StatefulProcessorNode<K, ?> statefulProcessorNode = getStatefulProcessorNode(
|
||||
named.suffixWithOrElseGet(
|
||||
"-cogroup-agg-" + counter++,
|
||||
builder,
|
||||
CogroupedKStreamImpl.AGGREGATE_NAME),
|
||||
stateCreated,
|
||||
storeBuilder,
|
||||
new KStreamSessionWindowAggregate<>(sessionWindows, storeBuilder.name(), initializer, kGroupedStream.getValue(), sessionMerger));
|
||||
stateCreated = true;
|
||||
processors.add(statefulProcessorNode);
|
||||
builder.addGraphNode(parentNodes.get(kGroupedStream.getKey()), statefulProcessorNode);
|
||||
}
|
||||
return createTable(processors, named, keySerde, valueSerde, queryableName);
|
||||
}
|
||||
|
||||
private void processRepartitions(final Map<KGroupedStreamImpl<K, ?>, Aggregator<? super K, ? super Object, VOut>> groupPatterns,
|
||||
final StoreBuilder<?> storeBuilder) {
|
||||
for (final KGroupedStreamImpl<K, ?> repartitionReqs : groupPatterns.keySet()) {
|
||||
|
||||
if (repartitionReqs.repartitionRequired) {
|
||||
|
@ -85,26 +160,14 @@ class CogroupedStreamAggregateBuilder<K, VOut> {
|
|||
groupedStreams.remove(kGrouped);
|
||||
kGrouped.ensureCopartitionWith(groupedStreams);
|
||||
|
||||
final Collection<StreamsGraphNode> processors = new ArrayList<>();
|
||||
boolean stateCreated = false;
|
||||
int counter = 0;
|
||||
for (final Entry<KGroupedStreamImpl<K, ?>, Aggregator<? super K, Object, VOut>> kGroupedStream : groupPatterns.entrySet()) {
|
||||
final StatefulProcessorNode<K, ?> statefulProcessorNode = getStatefulProcessorNode(
|
||||
kGroupedStream.getValue(),
|
||||
initializer,
|
||||
named.suffixWithOrElseGet(
|
||||
"-cogroup-agg-" + counter++,
|
||||
builder,
|
||||
CogroupedKStreamImpl.AGGREGATE_NAME),
|
||||
stateCreated,
|
||||
storeBuilder,
|
||||
windows,
|
||||
sessionWindows,
|
||||
sessionMerger);
|
||||
stateCreated = true;
|
||||
processors.add(statefulProcessorNode);
|
||||
builder.addGraphNode(parentNodes.get(kGroupedStream.getKey()), statefulProcessorNode);
|
||||
}
|
||||
|
||||
<KR, VIn> KTable<KR, VOut> createTable(final Collection<StreamsGraphNode> processors,
|
||||
final NamedInternal named,
|
||||
final Serde<KR> keySerde,
|
||||
final Serde<VOut> valueSerde,
|
||||
final String queryableName) {
|
||||
|
||||
final String mergeProcessorName = named.suffixWithOrElseGet(
|
||||
"-cogroup-merge",
|
||||
builder,
|
||||
|
@ -126,27 +189,10 @@ class CogroupedStreamAggregateBuilder<K, VOut> {
|
|||
builder);
|
||||
}
|
||||
|
||||
private <W extends Window> StatefulProcessorNode<K, ?> getStatefulProcessorNode(final Aggregator<? super K, Object, VOut> aggregator,
|
||||
final Initializer<VOut> initializer,
|
||||
final String processorName,
|
||||
private StatefulProcessorNode<K, ?> getStatefulProcessorNode(final String processorName,
|
||||
final boolean stateCreated,
|
||||
final StoreBuilder<?> storeBuilder,
|
||||
final Windows<W> windows,
|
||||
final SessionWindows sessionWindows,
|
||||
final Merger<? super K, VOut> sessionMerger) {
|
||||
|
||||
final ProcessorSupplier<K, ?> kStreamAggregate;
|
||||
|
||||
if (windows == null && sessionWindows == null) {
|
||||
kStreamAggregate = new KStreamAggregate<>(storeBuilder.name(), initializer, aggregator);
|
||||
} else if (windows != null && sessionWindows == null) {
|
||||
kStreamAggregate = new KStreamWindowAggregate<>(windows, storeBuilder.name(), initializer, aggregator);
|
||||
} else if (windows == null && sessionMerger != null) {
|
||||
kStreamAggregate = new KStreamSessionWindowAggregate<>(sessionWindows, storeBuilder.name(), initializer, aggregator, sessionMerger);
|
||||
} else {
|
||||
throw new IllegalArgumentException("must include windows OR sessionWindows + sessionMerger OR all must be null");
|
||||
}
|
||||
|
||||
final ProcessorSupplier<K, ?> kStreamAggregate) {
|
||||
final StatefulProcessorNode<K, ?> statefulProcessorNode;
|
||||
if (!stateCreated) {
|
||||
statefulProcessorNode =
|
||||
|
|
|
@ -91,7 +91,8 @@ public class SessionWindowedCogroupedKStreamImpl<K, V> extends
|
|||
materialized,
|
||||
builder,
|
||||
CogroupedKStreamImpl.AGGREGATE_NAME);
|
||||
return aggregateBuilder.build(groupPatterns,
|
||||
return aggregateBuilder.build(
|
||||
groupPatterns,
|
||||
initializer,
|
||||
new NamedInternal(named),
|
||||
materialize(materializedInternal),
|
||||
|
@ -101,7 +102,6 @@ public class SessionWindowedCogroupedKStreamImpl<K, V> extends
|
|||
null,
|
||||
materializedInternal.valueSerde(),
|
||||
materializedInternal.queryableStoreName(),
|
||||
null,
|
||||
sessionWindows,
|
||||
sessionMerger);
|
||||
}
|
||||
|
|
|
@ -100,9 +100,7 @@ public class TimeWindowedCogroupedKStreamImpl<K, V, W extends Window> extends Ab
|
|||
: null,
|
||||
materializedInternal.valueSerde(),
|
||||
materializedInternal.queryableStoreName(),
|
||||
windows,
|
||||
null,
|
||||
null);
|
||||
windows);
|
||||
}
|
||||
|
||||
@SuppressWarnings("deprecation")
|
||||
|
|
Loading…
Reference in New Issue