V
- value typeVR
- transformed value typepublic interface ValueTransformer<V,VR>
ValueTransformer
interface for stateful mapping of a value to a new value (with possible new type).
This is a stateful record-by-record operation, i.e, transform(Object)
is invoked individually for each
record of a stream and can access and modify a state that is available beyond a single call of
transform(Object)
(cf. ValueMapper
for stateless value transformation).
Additionally, this ValueTransformer
can schedule
a method to be called periodically
with the provided context.
If ValueTransformer
is applied to a KeyValue
pair record the record's key is preserved.
Use ValueTransformerSupplier
to provide new instances of ValueTransformer
to Kafka Stream's runtime.
If a record's key and value should be modified Transformer
can be used.
Modifier and Type | Method | Description |
---|---|---|
void |
close() |
Close this transformer and clean up any resources.
|
void |
init(ProcessorContext context) |
Initialize this transformer.
|
VR |
transform(V value) |
Transform the given value to a new value.
|
void init(ProcessorContext context)
close()
will be called on it; the
framework may later re-use the transformer by calling init(ProcessorContext)
again.
The provided context
can be used to access topology and record meta data, to
schedule
a method to be
called periodically
and to access attached StateStore
s.
Note that ProcessorContext
is updated in the background with the current record's meta data.
Thus, it only contains valid record meta data when accessed within transform(Object)
.
Note that using ProcessorContext.forward(Object, Object)
or
ProcessorContext.forward(Object, Object, To)
is not allowed within any method of
ValueTransformer
and will result in an exception
.
context
- the contextjava.lang.IllegalStateException
- If store gets registered after initialization is already finishedStreamsException
- if the store's change log does not contain the partitionVR transform(V value)
StateStore
that is attached
to this operator can be accessed and modified arbitrarily (cf.
ProcessorContext.getStateStore(String)
).
Note, that using ProcessorContext.forward(Object, Object)
or
ProcessorContext.forward(Object, Object, To)
is not allowed within transform
and
will result in an exception
.
value
- the value to be transformedvoid close()
init(ProcessorContext)
on it again.
It is not possible to return any new output records within close()
.
Using ProcessorContext.forward(Object, Object)
or ProcessorContext.forward(Object, Object, To)
will result in an exception
.