@InterfaceStability.Unstable public class PairStream<K,V> extends Stream<Pair<K,V>>
Represents a stream of key-value pairs.
Modifier and Type | Method and Description |
---|---|
<A,R> PairStream<K,R> |
aggregateByKey(CombinerAggregator<? super V,A,? extends R> aggregator)
Aggregates the values for each key of this stream using the given
CombinerAggregator . |
<R> PairStream<K,R> |
aggregateByKey(R initialValue,
BiFunction<? super R,? super V,? extends R> accumulator,
BiFunction<? super R,? super R,? extends R> combiner)
Aggregates the values for each key of this stream using the given initial value, accumulator and combiner.
|
PairStream<K,V>[] |
branch(Predicate<? super Pair<K,V>>... predicates)
Returns an array of streams by splitting the given stream into multiple branches based on the given predicates.
|
<V1> PairStream<K,Pair<Iterable<V>,Iterable<V1>>> |
coGroupByKey(PairStream<K,V1> otherStream)
Groups the values of this stream with the values having the same key from the other stream.
|
PairStream<K,Long> |
countByKey()
Counts the values for each key of this stream.
|
PairStream<K,V> |
filter(Predicate<? super Pair<K,V>> predicate)
Returns a stream consisting of the elements of this stream that matches the given filter.
|
<R> PairStream<K,R> |
flatMapValues(FlatMapFunction<? super V,? extends R> function)
Return a new stream by applying a
FlatMapFunction function to the value of each key-value pairs in this stream. |
<V1> PairStream<K,Pair<V,V1>> |
fullOuterJoin(PairStream<K,V1> otherStream)
Does a full outer join of the values of this stream with the values having the same key from the other stream.
|
<R,V1> PairStream<K,R> |
fullOuterJoin(PairStream<K,V1> otherStream,
ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Does a full outer join of the values of this stream with the values having the same key from the other stream.
|
PairStream<K,Iterable<V>> |
groupByKey()
Returns a new stream where the values are grouped by the keys.
|
PairStream<K,Iterable<V>> |
groupByKeyAndWindow(Window<?,?> window)
Returns a new stream where the values are grouped by keys and the given window.
|
<V1> PairStream<K,Pair<V,V1>> |
join(PairStream<K,V1> otherStream)
Join the values of this stream with the values having the same key from the other stream.
|
<R,V1> PairStream<K,R> |
join(PairStream<K,V1> otherStream,
ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Join the values of this stream with the values having the same key from the other stream.
|
<V1> PairStream<K,Pair<V,V1>> |
leftOuterJoin(PairStream<K,V1> otherStream)
Does a left outer join of the values of this stream with the values having the same key from the other stream.
|
<R,V1> PairStream<K,R> |
leftOuterJoin(PairStream<K,V1> otherStream,
ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Does a left outer join of the values of this stream with the values having the same key from the other stream.
|
<R> PairStream<K,R> |
mapValues(Function<? super V,? extends R> function)
Returns a new stream by applying a
Function to the value of each key-value pairs in this stream. |
PairStream<K,V> |
peek(Consumer<? super Pair<K,V>> action)
Returns a stream consisting of the elements of this stream, additionally performing the provided action on each element as they are consumed from the resulting stream.
|
PairStream<K,V> |
reduceByKey(Reducer<V> reducer)
Performs a reduction on the values for each key of this stream by repeatedly applying the reducer.
|
PairStream<K,V> |
reduceByKeyAndWindow(Reducer<V> reducer,
Window<?,?> window)
Returns a new stream where the values that arrive within a window having the same key will be reduced by repeatedly applying the reducer.
|
PairStream<K,V> |
repartition(int parallelism)
Returns a new stream with the given value of parallelism.
|
<V1> PairStream<K,Pair<V,V1>> |
rightOuterJoin(PairStream<K,V1> otherStream)
Does a right outer join of the values of this stream with the values having the same key from the other stream.
|
<R,V1> PairStream<K,R> |
rightOuterJoin(PairStream<K,V1> otherStream,
ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Does a right outer join of the values of this stream with the values having the same key from the other stream.
|
<R> StreamState<K,R> |
updateStateByKey(R initialValue,
BiFunction<? super R,? super V,? extends R> stateUpdateFn)
Update the state by applying the given state update function to the previous state of the key and the new value for the key.
|
<R> StreamState<K,R> |
updateStateByKey(StateUpdater<? super V,? extends R> stateUpdater)
Update the state by applying the given state update function to the previous state of the key and the new value for the key.
|
PairStream<K,V> |
window(Window<?,?> window)
Returns a new stream consisting of the elements that fall within the window as specified by the window parameter.
|
public <R> PairStream<K,R> mapValues(Function<? super V,? extends R> function)
Returns a new stream by applying a Function
to the value of each key-value pairs in this stream.
function
- the mapping functionR
- the result typepublic <R> PairStream<K,R> flatMapValues(FlatMapFunction<? super V,? extends R> function)
Return a new stream by applying a FlatMapFunction
function to the value of each key-value pairs in this stream.
function
- the flatmap functionR
- the result typepublic <R> PairStream<K,R> aggregateByKey(R initialValue, BiFunction<? super R,? super V,? extends R> accumulator, BiFunction<? super R,? super R,? extends R> combiner)
Aggregates the values for each key of this stream using the given initial value, accumulator and combiner.
initialValue
- the initial value of the resultaccumulator
- the accumulatorcombiner
- the combinerR
- the result typepublic <A,R> PairStream<K,R> aggregateByKey(CombinerAggregator<? super V,A,? extends R> aggregator)
Aggregates the values for each key of this stream using the given CombinerAggregator
.
aggregator
- the combiner aggregatorA
- the accumulator typeR
- the result typepublic PairStream<K,Long> countByKey()
Counts the values for each key of this stream.
public PairStream<K,V> reduceByKey(Reducer<V> reducer)
Performs a reduction on the values for each key of this stream by repeatedly applying the reducer.
reducer
- the reducerpublic PairStream<K,Iterable<V>> groupByKey()
Returns a new stream where the values are grouped by the keys.
public PairStream<K,Iterable<V>> groupByKeyAndWindow(Window<?,?> window)
Returns a new stream where the values are grouped by keys and the given window. The values that arrive within a window having the same key will be merged together and returned as an Iterable of values mapped to the key.
window
- the window configurationpublic PairStream<K,V> reduceByKeyAndWindow(Reducer<V> reducer, Window<?,?> window)
Returns a new stream where the values that arrive within a window having the same key will be reduced by repeatedly applying the reducer.
reducer
- the reducerwindow
- the window configurationpublic PairStream<K,V> peek(Consumer<? super Pair<K,V>> action)
Returns a stream consisting of the elements of this stream, additionally performing the provided action on each element as they are consumed from the resulting stream.
public PairStream<K,V> filter(Predicate<? super Pair<K,V>> predicate)
Returns a stream consisting of the elements of this stream that matches the given filter.
public <V1> PairStream<K,Pair<V,V1>> join(PairStream<K,V1> otherStream)
Join the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamV1
- the type of the values in the other streampublic <R,V1> PairStream<K,R> join(PairStream<K,V1> otherStream, ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Join the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamvalueJoiner
- the ValueJoiner
R
- the type of the values resulting from the joinV1
- the type of the values in the other streampublic <V1> PairStream<K,Pair<V,V1>> leftOuterJoin(PairStream<K,V1> otherStream)
Does a left outer join of the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamV1
- the type of the values in the other streampublic <R,V1> PairStream<K,R> leftOuterJoin(PairStream<K,V1> otherStream, ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Does a left outer join of the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamvalueJoiner
- the ValueJoiner
R
- the type of the values resulting from the joinV1
- the type of the values in the other streampublic <V1> PairStream<K,Pair<V,V1>> rightOuterJoin(PairStream<K,V1> otherStream)
Does a right outer join of the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamV1
- the type of the values in the other streampublic <R,V1> PairStream<K,R> rightOuterJoin(PairStream<K,V1> otherStream, ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Does a right outer join of the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamvalueJoiner
- the ValueJoiner
R
- the type of the values resulting from the joinV1
- the type of the values in the other streampublic <V1> PairStream<K,Pair<V,V1>> fullOuterJoin(PairStream<K,V1> otherStream)
Does a full outer join of the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamV1
- the type of the values in the other streampublic <R,V1> PairStream<K,R> fullOuterJoin(PairStream<K,V1> otherStream, ValueJoiner<? super V,? super V1,? extends R> valueJoiner)
Does a full outer join of the values of this stream with the values having the same key from the other stream.
Note: The parallelism of this stream is carried forward to the joined stream.
otherStream
- the other streamvalueJoiner
- the ValueJoiner
R
- the type of the values resulting from the joinV1
- the type of the values in the other streampublic PairStream<K,V> window(Window<?,?> window)
Returns a new stream consisting of the elements that fall within the window as specified by the window parameter. The Window
specification could be used to specify sliding or tumbling windows based on time duration or event count. For example,
// time duration based sliding window stream.window(SlidingWindows.of(Duration.minutes(10), Duration.minutes(1));// count based sliding window stream.window(SlidingWindows.of(Count.(10), Count.of(2)));
// time duration based tumbling window stream.window(TumblingWindows.of(Duration.seconds(10));
window
in class Stream<Pair<K,V>>
window
- the window configurationSlidingWindows
,
TumblingWindows
public PairStream<K,V> repartition(int parallelism)
Returns a new stream with the given value of parallelism. Further operations on this stream would execute at this level of parallelism.
repartition
in class Stream<Pair<K,V>>
parallelism
- the parallelism valuepublic PairStream<K,V>[] branch(Predicate<? super Pair<K,V>>... predicates)
Returns an array of streams by splitting the given stream into multiple branches based on the given predicates. The predicates are applied in the given order to the values of this stream and the result is forwarded to the corresponding (index based) result stream based on the (index of) predicate that matches.
Note: If none of the predicates match a value, that value is dropped.
public <R> StreamState<K,R> updateStateByKey(R initialValue, BiFunction<? super R,? super V,? extends R> stateUpdateFn)
Update the state by applying the given state update function to the previous state of the key and the new value for the key. This internally uses IStatefulBolt
to save the state. Use Config.TOPOLOGY_STATE_PROVIDER
to choose the state implementation.
stateUpdateFn
- the state update functionR
- the result typeStreamState
which can be used to query the statepublic <R> StreamState<K,R> updateStateByKey(StateUpdater<? super V,? extends R> stateUpdater)
Update the state by applying the given state update function to the previous state of the key and the new value for the key. This internally uses IStatefulBolt
to save the state. Use Config.TOPOLOGY_STATE_PROVIDER
to choose the state implementation.
stateUpdater
- the state updaterR
- the result typeStreamState
which can be used to query the statepublic <V1> PairStream<K,Pair<Iterable<V>,Iterable<V1>>> coGroupByKey(PairStream<K,V1> otherStream)
Groups the values of this stream with the values having the same key from the other stream.
If stream1 has values - (k1, v1), (k2, v2), (k2, v3)
and stream2 has values - (k1, x1), (k1, x2), (k3, x3)
The the co-grouped stream would contain - (k1, ([v1], [x1, x2]), (k2, ([v2, v3], [])), (k3, ([], [x3]))
Note: The parallelism of this stream is carried forward to the co-grouped stream.
otherStream
- the other streamV1
- the type of the values in the other streamCopyright © 2022 The Apache Software Foundation. All rights reserved.