Interface Transformer<K,V,R>
-
- Type Parameters:
K
- key typeV
- value typeR
-KeyValue
return type (both key and value type can be set arbitrarily)
public interface Transformer<K,V,R>
TheTransformer
interface is for stateful mapping of an input record to zero, one, or multiple new output records (both key and value type can be altered arbitrarily). This is a stateful record-by-record operation, i.e,transform(Object, Object)
is invoked individually for each record of a stream and can access and modify a state that is available beyond a single call oftransform(Object, Object)
(cf.KeyValueMapper
for stateless record transformation). Additionally, thisTransformer
canschedule
a method to becalled periodically
with the provided context.Use
TransformerSupplier
to provide new instances ofTransformer
to Kafka Stream's runtime.If only a record's value should be modified
ValueTransformer
can be used.
-
-
Method Summary
All Methods Instance Methods Abstract Methods Modifier and Type Method Description void
close()
Close this transformer and clean up any resources.void
init(ProcessorContext context)
Initialize this transformer.R
transform(K key, V value)
Transform the record with the given key and value.
-
-
-
Method Detail
-
init
void init(ProcessorContext context)
Initialize this transformer. This is called once per instance when the topology gets initialized. When the framework is done with the transformer,close()
will be called on it; the framework may later re-use the transformer by callinginit(ProcessorContext)
again.The provided
context
can be used to access topology and record meta data, toschedule
a method to becalled periodically
and to access attachedStateStore
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 withintransform(Object, Object)
.- Parameters:
context
- the context
-
transform
R transform(K key, V value)
Transform the record with the given key and value. Additionally, anystate
that isattached
to this operator can be accessed and modified arbitrarily (cf.ProcessorContext.getStateStore(String)
).If only one record should be forward downstream,
transform
can return a newKeyValue
. If more than one output record should be forwarded downstream,ProcessorContext.forward(Object, Object)
andProcessorContext.forward(Object, Object, To)
can be used. If no record should be forwarded downstream,transform
can returnnull
. Note that returning a newKeyValue
is merely for convenience. The same can be achieved by usingProcessorContext.forward(Object, Object)
and returningnull
.- Parameters:
key
- the key for the recordvalue
- the value for the record- Returns:
- new
KeyValue
pair—ifnull
no key-value pair will be forwarded to down stream
-
close
void close()
Close this transformer and clean up any resources. The framework may later re-use this transformer by callinginit(ProcessorContext)
on it again.To generate new
KeyValue
pairsProcessorContext.forward(Object, Object)
andProcessorContext.forward(Object, Object, To)
can be used.
-
-