public abstract class StateSpec<KeyType,ValueType,StateType,MappedType>
extends java.lang.Object
implements scala.Serializable
mapWithState
operation of a
pair DStream
(Scala) or a
JavaPairDStream
(Java).
Use StateSpec.function
factory methods
to create instances of this class.
Example in Scala:
// A mapping function that maintains an integer state and return a String
def mappingFunction(key: String, value: Option[Int], state: State[Int]): Option[String] = {
// Use state.exists(), state.get(), state.update() and state.remove()
// to manage state, and return the necessary string
}
val spec = StateSpec.function(mappingFunction).numPartitions(10)
val mapWithStateDStream = keyValueDStream.mapWithState[StateType, MappedType](spec)
Example in Java:
// A mapping function that maintains an integer state and return a string
Function3<String, Optional<Integer>, State<Integer>, String> mappingFunction =
new Function3<String, Optional<Integer>, State<Integer>, String>() {
@Override
public Optional<String> call(Optional<Integer> value, State<Integer> state) {
// Use state.exists(), state.get(), state.update() and state.remove()
// to manage state, and return the necessary string
}
};
JavaMapWithStateDStream<String, Integer, Integer, String> mapWithStateDStream =
keyValueDStream.mapWithState(StateSpec.function(mappingFunc));
Constructor and Description |
---|
StateSpec() |
Modifier and Type | Method and Description |
---|---|
static <KeyType,ValueType,StateType,MappedType> |
function(scala.Function3<KeyType,scala.Option<ValueType>,State<StateType>,MappedType> mappingFunction)
Create a
StateSpec for setting all the specifications
of the mapWithState operation on a
pair DStream . |
static <KeyType,ValueType,StateType,MappedType> |
function(Function3<KeyType,Optional<ValueType>,State<StateType>,MappedType> mappingFunction)
Create a
StateSpec for setting all the specifications
of the mapWithState operation on a
JavaPairDStream . |
static <KeyType,ValueType,StateType,MappedType> |
function(scala.Function4<Time,KeyType,scala.Option<ValueType>,State<StateType>,scala.Option<MappedType>> mappingFunction)
Create a
StateSpec for setting all the specifications
of the mapWithState operation on a
pair DStream . |
static <KeyType,ValueType,StateType,MappedType> |
function(Function4<Time,KeyType,Optional<ValueType>,State<StateType>,Optional<MappedType>> mappingFunction)
Create a
StateSpec for setting all
the specifications of the mapWithState operation on a
JavaPairDStream . |
abstract StateSpec<KeyType,ValueType,StateType,MappedType> |
initialState(JavaPairRDD<KeyType,StateType> javaPairRDD)
Set the RDD containing the initial states that will be used by `mapWithState`
|
abstract StateSpec<KeyType,ValueType,StateType,MappedType> |
initialState(RDD<scala.Tuple2<KeyType,StateType>> rdd)
Set the RDD containing the initial states that will be used by `mapWithState`
|
abstract StateSpec<KeyType,ValueType,StateType,MappedType> |
numPartitions(int numPartitions)
Set the number of partitions by which the state RDDs generated by
mapWithState
will be partitioned. |
abstract StateSpec<KeyType,ValueType,StateType,MappedType> |
partitioner(Partitioner partitioner)
Set the partitioner by which the state RDDs generated by
mapWithState will be partitioned. |
abstract StateSpec<KeyType,ValueType,StateType,MappedType> |
timeout(Duration idleDuration)
Set the duration after which the state of an idle key will be removed.
|
public static <KeyType,ValueType,StateType,MappedType> StateSpec<KeyType,ValueType,StateType,MappedType> function(scala.Function4<Time,KeyType,scala.Option<ValueType>,State<StateType>,scala.Option<MappedType>> mappingFunction)
StateSpec
for setting all the specifications
of the mapWithState
operation on a
pair DStream
.
mappingFunction
- The function applied on every data item to manage the associated state
and generate the mapped datapublic static <KeyType,ValueType,StateType,MappedType> StateSpec<KeyType,ValueType,StateType,MappedType> function(scala.Function3<KeyType,scala.Option<ValueType>,State<StateType>,MappedType> mappingFunction)
StateSpec
for setting all the specifications
of the mapWithState
operation on a
pair DStream
.
mappingFunction
- The function applied on every data item to manage the associated state
and generate the mapped datapublic static <KeyType,ValueType,StateType,MappedType> StateSpec<KeyType,ValueType,StateType,MappedType> function(Function4<Time,KeyType,Optional<ValueType>,State<StateType>,Optional<MappedType>> mappingFunction)
StateSpec
for setting all
the specifications of the mapWithState
operation on a
JavaPairDStream
.
mappingFunction
- The function applied on every data item to manage the associated
state and generate the mapped datapublic static <KeyType,ValueType,StateType,MappedType> StateSpec<KeyType,ValueType,StateType,MappedType> function(Function3<KeyType,Optional<ValueType>,State<StateType>,MappedType> mappingFunction)
StateSpec
for setting all the specifications
of the mapWithState
operation on a
JavaPairDStream
.
mappingFunction
- The function applied on every data item to manage the associated
state and generate the mapped datapublic abstract StateSpec<KeyType,ValueType,StateType,MappedType> initialState(RDD<scala.Tuple2<KeyType,StateType>> rdd)
public abstract StateSpec<KeyType,ValueType,StateType,MappedType> initialState(JavaPairRDD<KeyType,StateType> javaPairRDD)
public abstract StateSpec<KeyType,ValueType,StateType,MappedType> numPartitions(int numPartitions)
mapWithState
will be partitioned. Hash partitioning will be used.numPartitions
- (undocumented)public abstract StateSpec<KeyType,ValueType,StateType,MappedType> partitioner(Partitioner partitioner)
mapWithState
will be partitioned.partitioner
- (undocumented)public abstract StateSpec<KeyType,ValueType,StateType,MappedType> timeout(Duration idleDuration)
State.isTimingOut()
set
to true
in that call.idleDuration
- (undocumented)