http://git-wip-us.apache.org/repos/asf/storm/blob/d491c3ff/docs/Transactional-topologies.md ---------------------------------------------------------------------- diff --git a/docs/Transactional-topologies.md b/docs/Transactional-topologies.md new file mode 100644 index 0000000..1271a21 --- /dev/null +++ b/docs/Transactional-topologies.md @@ -0,0 +1,359 @@ +--- +layout: documentation +--- +**NOTE**: Transactional topologies have been deprecated -- use the [Trident](Trident-tutorial.html) framework instead. + +__________________________________________________________________________ + +Storm [guarantees data processing](Guaranteeing-message-processing.html) by providing an at least once processing guarantee. The most common question asked about Storm is "Given that tuples can be replayed, how do you do things like counting on top of Storm? Won't you overcount?" + +Storm 0.7.0 introduces transactional topologies, which enable you to get exactly once messaging semantics for pretty much any computation. So you can do things like counting in a fully-accurate, scalable, and fault-tolerant way. + +Like [Distributed RPC](Distributed-RPC.html), transactional topologies aren't so much a feature of Storm as they are a higher level abstraction built on top of Storm's primitives of streams, spouts, bolts, and topologies. + +This page explains the transactional topology abstraction, how to use the API, and provides details as to its implementation. + +## Concepts + +Let's build up to Storm's abstraction for transactional topologies one step at a time. Let's start by looking at the simplest possible approach, and then we'll iterate on the design until we reach Storm's design. + +### Design 1 + +The core idea behind transactional topologies is to provide a _strong ordering_ on the processing of data. The simplest manifestation of this, and the first design we'll look at, is processing the tuples one at a time and not moving on to the next tuple until the current tuple has been successfully processed by the topology. + +Each tuple is associated with a transaction id. If the tuple fails and needs to be replayed, then it is emitted with the exact same transaction id. A transaction id is an integer that increments for every tuple, so the first tuple will have transaction id `1`, the second id `2`, and so on. + +The strong ordering of tuples gives you the capability to achieve exactly-once semantics even in the case of tuple replay. Let's look at an example of how you would do this. + +Suppose you want to do a global count of the tuples in the stream. Instead of storing just the count in the database, you instead store the count and the latest transaction id together as one value in the database. When your code updates the count in the db, it should update the count *only if the transaction id in the database differs from the transaction id for the tuple currently being processed*. Consider the two cases: + +1. *The transaction id in the database is different than the current transaction id:* Because of the strong ordering of transactions, we know for sure that the current tuple isn't represented in that count. So we can safely increment the count and update the transaction id. +2. *The transaction id is the same as the current transaction id:* Then we know that this tuple is already incorporated into the count and can skip the update. The tuple must have failed after updating the database but before reporting success back to Storm. + +This logic and the strong ordering of transactions ensures that the count in the database will be accurate even if tuples are replayed. Credit for this trick of storing a transaction id in the database along with the value goes to the Kafka devs, particularly [this design document](http://incubator.apache.org/kafka/07/design.html). + +Furthermore, notice that the topology can safely update many sources of state in the same transaction and achieve exactly-once semantics. If there's a failure, any updates that already succeeded will skip on the retry, and any updates that failed will properly retry. For example, if you were processing a stream of tweeted urls, you could update a database that stores a tweet count for each url as well as a database that stores a tweet count for each domain. + +There is a significant problem though with this design of processing one tuple at time. Having to wait for each tuple to be _completely processed_ before moving on to the next one is horribly inefficient. It entails a huge amount of database calls (at least one per tuple), and this design makes very little use of the parallelization capabilities of Storm. So it isn't very scalable. + +### Design 2 + +Instead of processing one tuple at a time, a better approach is to process a batch of tuples for each transaction. So if you're doing a global count, you would increment the count by the number of tuples in the entire batch. If a batch fails, you replay the exact batch that failed. Instead of assigning a transaction id to each tuple, you assign a transaction id to each batch, and the processing of the batches is strongly ordered. Here's a diagram of this design: + + + +So if you're processing 1000 tuples per batch, your application will do 1000x less database operations than design 1. Additionally, it takes advantage of Storm's parallelization capabilities as the computation for each batch can be parallelized. + +While this design is significantly better than design 1, it's still not as resource-efficient as possible. The workers in the topology spend a lot of time being idle waiting for the other portions of the computation to finish. For example, in a topology like this: + + + +After bolt 1 finishes its portion of the processing, it will be idle until the rest of the bolts finish and the next batch can be emitted from the spout. + +### Design 3 (Storm's design) + +A key realization is that not all the work for processing batches of tuples needs to be strongly ordered. For example, when computing a global count, there's two parts to the computation: + +1. Computing the partial count for the batch +2. Updating the global count in the database with the partial count + +The computation of #2 needs to be strongly ordered across the batches, but there's no reason you shouldn't be able to _pipeline_ the computation of the batches by computing #1 for many batches in parallel. So while batch 1 is working on updating the database, batches 2 through 10 can compute their partial counts. + +Storm accomplishes this distinction by breaking the computation of a batch into two phases: + +1. The processing phase: this is the phase that can be done in parallel for many batches +2. The commit phase: The commit phases for batches are strongly ordered. So the commit for batch 2 is not done until the commit for batch 1 has been successful. + +The two phases together are called a "transaction". Many batches can be in the processing phase at a given moment, but only one batch can be in the commit phase. If there's any failure in the processing or commit phase for a batch, the entire transaction is replayed (both phases). + +## Design details + +When using transactional topologies, Storm does the following for you: + +1. *Manages state:* Storm stores in Zookeeper all the state necessary to do transactional topologies. This includes the current transaction id as well as the metadata defining the parameters for each batch. +2. *Coordinates the transactions:* Storm will manage everything necessary to determine which transactions should be processing or committing at any point. +3. *Fault detection:* Storm leverages the acking framework to efficiently determine when a batch has successfully processed, successfully committed, or failed. Storm will then replay batches appropriately. You don't have to do any acking or anchoring -- Storm manages all of this for you. +4. *First class batch processing API*: Storm layers an API on top of regular bolts to allow for batch processing of tuples. Storm manages all the coordination for determining when a task has received all the tuples for that particular transaction. Storm will also take care of cleaning up any accumulated state for each transaction (like the partial counts). + +Finally, another thing to note is that transactional topologies require a source queue that can replay an exact batch of messages. Technologies like [Kestrel](https://github.com/robey/kestrel) can't do this. [Apache Kafka](http://incubator.apache.org/kafka/index.html) is a perfect fit for this kind of spout, and [storm-kafka](https://github.com/nathanmarz/storm-contrib/tree/master/storm-kafka) in [storm-contrib](https://github.com/nathanmarz/storm-contrib) contains a transactional spout implementation for Kafka. + +## The basics through example + +You build transactional topologies by using [TransactionalTopologyBuilder](javadocs/backtype/storm/transactional/TransactionalTopologyBuilder.html). Here's the transactional topology definition for a topology that computes the global count of tuples from the input stream. This code comes from [TransactionalGlobalCount](https://github.com/nathanmarz/storm-starter/blob/master/src/jvm/storm/starter/TransactionalGlobalCount.java) in storm-starter. + +```java +MemoryTransactionalSpout spout = new MemoryTransactionalSpout(DATA, new Fields("word"), PARTITION_TAKE_PER_BATCH); +TransactionalTopologyBuilder builder = new TransactionalTopologyBuilder("global-count", "spout", spout, 3); +builder.setBolt("partial-count", new BatchCount(), 5) + .shuffleGrouping("spout"); +builder.setBolt("sum", new UpdateGlobalCount()) + .globalGrouping("partial-count"); +``` + +`TransactionalTopologyBuilder` takes as input in the constructor an id for the transactional topology, an id for the spout within the topology, a transactional spout, and optionally the parallelism for the transactional spout. The id for the transactional topology is used to store state about the progress of topology in Zookeeper, so that if you restart the topology it will continue where it left off. + +A transactional topology has a single `TransactionalSpout` that is defined in the constructor of `TransactionalTopologyBuilder`. In this example, `MemoryTransactionalSpout` is used which reads in data from an in-memory partitioned source of data (the `DATA` variable). The second argument defines the fields for the data, and the third argument specifies the maximum number of tuples to emit from each partition per batch of tuples. The interface for defining your own transactional spouts is discussed later on in this tutorial. + +Now on to the bolts. This topology parallelizes the computation of the global count. The first bolt, `BatchCount`, randomly partitions the input stream using a shuffle grouping and emits the count for each partition. The second bolt, `UpdateGlobalCount`, does a global grouping and sums together the partial counts to get the count for the batch. It then updates the global count in the database if necessary. + +Here's the definition of `BatchCount`: + +```java +public static class BatchCount extends BaseBatchBolt { + Object _id; + BatchOutputCollector _collector; + + int _count = 0; + + @Override + public void prepare(Map conf, TopologyContext context, BatchOutputCollector collector, Object id) { + _collector = collector; + _id = id; + } + + @Override + public void execute(Tuple tuple) { + _count++; + } + + @Override + public void finishBatch() { + _collector.emit(new Values(_id, _count)); + } + + @Override + public void declareOutputFields(OutputFieldsDeclarer declarer) { + declarer.declare(new Fields("id", "count")); + } +} +``` + +A new instance of this object is created for every batch that's being processed. The actual bolt this runs within is called [BatchBoltExecutor](https://github.com/apache/incubator-storm/blob/0.7.0/src/jvm/backtype/storm/coordination/BatchBoltExecutor.java) and manages the creation and cleanup for these objects. + +The `prepare` method parameterizes this batch bolt with the Storm config, the topology context, an output collector, and the id for this batch of tuples. In the case of transactional topologies, the id will be a [TransactionAttempt](javadocs/backtype/storm/transactional/TransactionAttempt.html) object. The batch bolt abstraction can be used in Distributed RPC as well which uses a different type of id for the batches. `BatchBolt` can actually be parameterized with the type of the id, so if you only intend to use the batch bolt for transactional topologies, you can extend `BaseTransactionalBolt` which has this definition: + +```java +public abstract class BaseTransactionalBolt extends BaseBatchBolt<TransactionAttempt> { +} +``` + +All tuples emitted within a transactional topology must have the `TransactionAttempt` as the first field of the tuple. This lets Storm identify which tuples belong to which batches. So when you emit tuples you need to make sure to meet this requirement. + +The `TransactionAttempt` contains two values: the "transaction id" and the "attempt id". The "transaction id" is the unique id chosen for this batch and is the same no matter how many times the batch is replayed. The "attempt id" is a unique id for this particular batch of tuples and lets Storm distinguish tuples from different emissions of the same batch. Without the attempt id, Storm could confuse a replay of a batch with tuples from a prior time that batch was emitted. This would be disastrous. + +The transaction id increases by 1 for every batch emitted. So the first batch has id "1", the second has id "2", and so on. + +The `execute` method is called for every tuple in the batch. You should accumulate state for the batch in a local instance variable every time this method is called. The `BatchCount` bolt increments a local counter variable for every tuple. + +Finally, `finishBatch` is called when the task has received all tuples intended for it for this particular batch. `BatchCount` emits the partial count to the output stream when this method is called. + +Here's the definition of `UpdateGlobalCount`: + +```java +public static class UpdateGlobalCount extends BaseTransactionalBolt implements ICommitter { + TransactionAttempt _attempt; + BatchOutputCollector _collector; + + int _sum = 0; + + @Override + public void prepare(Map conf, TopologyContext context, BatchOutputCollector collector, TransactionAttempt attempt) { + _collector = collector; + _attempt = attempt; + } + + @Override + public void execute(Tuple tuple) { + _sum+=tuple.getInteger(1); + } + + @Override + public void finishBatch() { + Value val = DATABASE.get(GLOBAL_COUNT_KEY); + Value newval; + if(val == null || !val.txid.equals(_attempt.getTransactionId())) { + newval = new Value(); + newval.txid = _attempt.getTransactionId(); + if(val==null) { + newval.count = _sum; + } else { + newval.count = _sum + val.count; + } + DATABASE.put(GLOBAL_COUNT_KEY, newval); + } else { + newval = val; + } + _collector.emit(new Values(_attempt, newval.count)); + } + + @Override + public void declareOutputFields(OutputFieldsDeclarer declarer) { + declarer.declare(new Fields("id", "sum")); + } +} +``` + +`UpdateGlobalCount` is specific to transactional topologies so it extends `BaseTransactionalBolt`. In the `execute` method, `UpdateGlobalCount` accumulates the count for this batch by summing together the partial batches. The interesting stuff happens in `finishBatch`. + +First, notice that this bolt implements the `ICommitter` interface. This tells Storm that the `finishBatch` method of this bolt should be part of the commit phase of the transaction. So calls to `finishBatch` for this bolt will be strongly ordered by transaction id (calls to `execute` on the other hand can happen during either the processing or commit phases). An alternative way to mark a bolt as a committer is to use the `setCommitterBolt` method in `TransactionalTopologyBuilder` instead of `setBolt`. + +The code for `finishBatch` in `UpdateGlobalCount` gets the current value from the database and compares its transaction id to the transaction id for this batch. If they are the same, it does nothing. Otherwise, it increments the value in the database by the partial count for this batch. + +A more involved transactional topology example that updates multiple databases idempotently can be found in storm-starter in the [TransactionalWords](https://github.com/nathanmarz/storm-starter/blob/master/src/jvm/storm/starter/TransactionalWords.java) class. + +## Transactional Topology API + +This section outlines the different pieces of the transactional topology API. + +### Bolts + +There are three kinds of bolts possible in a transactional topology: + +1. [BasicBolt](javadocs/backtype/storm/topology/base/BaseBasicBolt.html): This bolt doesn't deal with batches of tuples and just emits tuples based on a single tuple of input. +2. [BatchBolt](javadocs/backtype/storm/topology/base/BaseBatchBolt.html): This bolt processes batches of tuples. `execute` is called for each tuple, and `finishBatch` is called when the batch is complete. +3. BatchBolt's that are marked as committers: The only difference between this bolt and a regular batch bolt is when `finishBatch` is called. A committer bolt has `finishedBatch` called during the commit phase. The commit phase is guaranteed to occur only after all prior batches have successfully committed, and it will be retried until all bolts in the topology succeed the commit for the batch. There are two ways to make a `BatchBolt` a committer, by having the `BatchBolt` implement the [ICommitter](javadocs/backtype/storm/transactional/ICommitter.html) marker interface, or by using the `setCommiterBolt` method in `TransactionalTopologyBuilder`. + +#### Processing phase vs. commit phase in bolts + +To nail down the difference between the processing phase and commit phase of a transaction, let's look at an example topology: + + + +In this topology, only the bolts with a red outline are committers. + +During the processing phase, bolt A will process the complete batch from the spout, call `finishBatch` and send its tuples to bolts B and C. Bolt B is a committer so it will process all the tuples but finishBatch won't be called. Bolt C also will not have `finishBatch` called because it doesn't know if it has received all the tuples from Bolt B yet (because Bolt B is waiting for the transaction to commit). Finally, Bolt D will receive any tuples Bolt C emitted during invocations of its `execute` method. + +When the batch commits, `finishBatch` is called on Bolt B. Once it finishes, Bolt C can now detect that it has received all the tuples and will call `finishBatch`. Finally, Bolt D will receive its complete batch and call `finishBatch`. + +Notice that even though Bolt D is a committer, it doesn't have to wait for a second commit message when it receives the whole batch. Since it receives the whole batch during the commit phase, it goes ahead and completes the transaction. + +Committer bolts act just like batch bolts during the commit phase. The only difference between committer bolts and batch bolts is that committer bolts will not call `finishBatch` during the processing phase of a transaction. + +#### Acking + +Notice that you don't have to do any acking or anchoring when working with transactional topologies. Storm manages all of that underneath the hood. The acking strategy is heavily optimized. + +#### Failing a transaction + +When using regular bolts, you can call the `fail` method on `OutputCollector` to fail the tuple trees of which that tuple is a member. Since transactional topologies hide the acking framework from you, they provide a different mechanism to fail a batch (and cause the batch to be replayed). Just throw a [FailedException](javadocs/backtype/storm/topology/FailedException.html). Unlike regular exceptions, this will only cause that particular batch to replay and will not crash the process. + +### Transactional spout + +The `TransactionalSpout` interface is completely different from a regular `Spout` interface. A `TransactionalSpout` implementation emits batches of tuples and must ensure that the same batch of tuples is always emitted for the same transaction id. + +A transactional spout looks like this while a topology is executing: + + + +The coordinator on the left is a regular Storm spout that emits a tuple whenever a batch should be emitted for a transaction. The emitters execute as a regular Storm bolt and are responsible for emitting the actual tuples for the batch. The emitters subscribe to the "batch emit" stream of the coordinator using an all grouping. + +The need to be idempotent with respect to the tuples it emits requires a `TransactionalSpout` to store a small amount of state. The state is stored in Zookeeper. + +The details of implementing a `TransactionalSpout` are in [the Javadoc](javadocs/backtype/storm/transactional/ITransactionalSpout.html). + +#### Partitioned Transactional Spout + +A common kind of transactional spout is one that reads the batches from a set of partitions across many queue brokers. For example, this is how [TransactionalKafkaSpout](https://github.com/nathanmarz/storm-contrib/blob/master/storm-kafka/src/jvm/storm/kafka/TransactionalKafkaSpout.java) works. An `IPartitionedTransactionalSpout` automates the bookkeeping work of managing the state for each partition to ensure idempotent replayability. See [the Javadoc](javadocs/backtype/storm/transactional/partitioned/IPartitionedTransactionalSpout.html) for more details. + +### Configuration + +There's two important bits of configuration for transactional topologies: + +1. *Zookeeper:* By default, transactional topologies will store state in the same Zookeeper instance as used to manage the Storm cluster. You can override this with the "transactional.zookeeper.servers" and "transactional.zookeeper.port" configs. +2. *Number of active batches permissible at once:* You must set a limit to the number of batches that can be processed at once. You configure this using the "topology.max.spout.pending" config. If you don't set this config, it will default to 1. + +## What if you can't emit the same batch of tuples for a given transaction id? + +So far the discussion around transactional topologies has assumed that you can always emit the exact same batch of tuples for the same transaction id. So what do you do if this is not possible? + +Consider an example of when this is not possible. Suppose you are reading tuples from a partitioned message broker (stream is partitioned across many machines), and a single transaction will include tuples from all the individual machines. Now suppose one of the nodes goes down at the same time that a transaction fails. Without that node, it is impossible to replay the same batch of tuples you just played for that transaction id. The processing in your topology will halt as its unable to replay the identical batch. The only possible solution is to emit a different batch for that transaction id than you emitted before. Is it possible to still achieve exactly-once messaging semantics even if the batches change? + +It turns out that you can still achieve exactly-once messaging semantics in your processing with a non-idempotent transactional spout, although this requires a bit more work on your part in developing the topology. + +If a batch can change for a given transaction id, then the logic we've been using so far of "skip the update if the transaction id in the database is the same as the id for the current transaction" is no longer valid. This is because the current batch is different than the batch for the last time the transaction was committed, so the result will not necessarily be the same. You can fix this problem by storing a little bit more state in the database. Let's again use the example of storing a global count in the database and suppose the partial count for the batch is stored in the `partialCount` variable. + +Instead of storing a value in the database that looks like this: + +```java +class Value { + Object count; + BigInteger txid; +} +``` + +For non-idempotent transactional spouts you should instead store a value that looks like this: + +```java +class Value { + Object count; + BigInteger txid; + Object prevCount; +} +``` + +The logic for the update is as follows: + +1. If the transaction id for the current batch is the same as the transaction id in the database, set `val.count = val.prevCount + partialCount`. +2. Otherwise, set `val.prevCount = val.count`, `val.count = val.count + partialCount` and `val.txid = batchTxid`. + +This logic works because once you commit a particular transaction id for the first time, all prior transaction ids will never be committed again. + +There's a few more subtle aspects of transactional topologies that make opaque transactional spouts possible. + +When a transaction fails, all subsequent transactions in the processing phase are considered failed as well. Each of those transactions will be re-emitted and reprocessed. Without this behavior, the following situation could happen: + +1. Transaction A emits tuples 1-50 +2. Transaction B emits tuples 51-100 +3. Transaction A fails +4. Transaction A emits tuples 1-40 +5. Transaction A commits +6. Transaction B commits +7. Transaction C emits tuples 101-150 + +In this scenario, tuples 41-50 are skipped. By failing all subsequent transactions, this would happen instead: + +1. Transaction A emits tuples 1-50 +2. Transaction B emits tuples 51-100 +3. Transaction A fails (and causes Transaction B to fail) +4. Transaction A emits tuples 1-40 +5. Transaction B emits tuples 41-90 +5. Transaction A commits +6. Transaction B commits +7. Transaction C emits tuples 91-140 + +By failing all subsequent transactions on failure, no tuples are skipped. This also shows that a requirement of transactional spouts is that they always emit where the last transaction left off. + +A non-idempotent transactional spout is more concisely referred to as an "OpaqueTransactionalSpout" (opaque is the opposite of idempotent). [IOpaquePartitionedTransactionalSpout](javadocs/backtype/storm/transactional/partitioned/IOpaquePartitionedTransactionalSpout.html) is an interface for implementing opaque partitioned transactional spouts, of which [OpaqueTransactionalKafkaSpout](https://github.com/nathanmarz/storm-contrib/blob/kafka0.7/storm-kafka/src/jvm/storm/kafka/OpaqueTransactionalKafkaSpout.java) is an example. `OpaqueTransactionalKafkaSpout` can withstand losing individual Kafka nodes without sacrificing accuracy as long as you use the update strategy as explained in this section. + +## Implementation + +The implementation for transactional topologies is very elegant. Managing the commit protocol, detecting failures, and pipelining batches seem complex, but everything turns out to be a straightforward mapping to Storm's primitives. + +How the data flow works: + +Here's how transactional spout works: + +1. Transactional spout is a subtopology consisting of a coordinator spout and an emitter bolt +2. The coordinator is a regular spout with a parallelism of 1 +3. The emitter is a bolt with a parallelism of P, connected to the coordinator's "batch" stream using an all grouping +4. When the coordinator determines it's time to enter the processing phase for a transaction, it emits a tuple containing the TransactionAttempt and the metadata for that transaction to the "batch" stream +5. Because of the all grouping, every single emitter task receives the notification that it's time to emit its portion of the tuples for that transaction attempt +6. Storm automatically manages the anchoring/acking necessary throughout the whole topology to determine when a transaction has completed the processing phase. The key here is that *the root tuple was created by the coordinator, so the coordinator will receive an "ack" if the processing phase succeeds, and a "fail" if it doesn't succeed for any reason (failure or timeout). +7. If the processing phase succeeds, and all prior transactions have successfully committed, the coordinator emits a tuple containing the TransactionAttempt to the "commit" stream. +8. All committing bolts subscribe to the commit stream using an all grouping, so that they will all receive a notification when the commit happens. +9. Like the processing phase, the coordinator uses the acking framework to determine whether the commit phase succeeded or not. If it receives an "ack", it marks that transaction as complete in zookeeper. + +More notes: + +- Transactional spouts are a sub-topology consisting of a spout and a bolt + - the spout is the coordinator and contains a single task + - the bolt is the emitter + - the bolt subscribes to the coordinator with an all grouping + - serialization of metadata is handled by kryo. kryo is initialized ONLY with the registrations defined in the component configuration for the transactionalspout +- the coordinator uses the acking framework to determine when a batch has been successfully processed, and then to determine when a batch has been successfully committed. +- state is stored in zookeeper using RotatingTransactionalState +- commiting bolts subscribe to the coordinators commit stream using an all grouping +- CoordinatedBolt is used to detect when a bolt has received all the tuples for a particular batch. + - this is the same abstraction that is used in DRPC + - for commiting bolts, it waits to receive a tuple from the coordinator's commit stream before calling finishbatch + - so it can't call finishbatch until it's received all tuples from all subscribed components AND its received the commit stream tuple (for committers). this ensures that it can't prematurely call finishBatch
http://git-wip-us.apache.org/repos/asf/storm/blob/d491c3ff/docs/Trident-API-Overview.md ---------------------------------------------------------------------- diff --git a/docs/Trident-API-Overview.md b/docs/Trident-API-Overview.md new file mode 100644 index 0000000..3b68645 --- /dev/null +++ b/docs/Trident-API-Overview.md @@ -0,0 +1,311 @@ +--- +layout: documentation +--- +# Trident API overview + +The core data model in Trident is the "Stream", processed as a series of batches. A stream is partitioned among the nodes in the cluster, and operations applied to a stream are applied in parallel across each partition. + +There are five kinds of operations in Trident: + +1. Operations that apply locally to each partition and cause no network transfer +2. Repartitioning operations that repartition a stream but otherwise don't change the contents (involves network transfer) +3. Aggregation operations that do network transfer as part of the operation +4. Operations on grouped streams +5. Merges and joins + +## Partition-local operations + +Partition-local operations involve no network transfer and are applied to each batch partition independently. + +### Functions + +A function takes in a set of input fields and emits zero or more tuples as output. The fields of the output tuple are appended to the original input tuple in the stream. If a function emits no tuples, the original input tuple is filtered out. Otherwise, the input tuple is duplicated for each output tuple. Suppose you have this function: + +```java +public class MyFunction extends BaseFunction { + public void execute(TridentTuple tuple, TridentCollector collector) { + for(int i=0; i < tuple.getInteger(0); i++) { + collector.emit(new Values(i)); + } + } +} +``` + +Now suppose you have a stream in the variable "mystream" with the fields ["a", "b", "c"] with the following tuples: + +``` +[1, 2, 3] +[4, 1, 6] +[3, 0, 8] +``` + +If you run this code: + +```java +mystream.each(new Fields("b"), new MyFunction(), new Fields("d"))) +``` + +The resulting tuples would have fields ["a", "b", "c", "d"] and look like this: + +``` +[1, 2, 3, 0] +[1, 2, 3, 1] +[4, 1, 6, 0] +``` + +### Filters + +Filters take in a tuple as input and decide whether or not to keep that tuple or not. Suppose you had this filter: + +```java +public class MyFilter extends BaseFunction { + public boolean isKeep(TridentTuple tuple) { + return tuple.getInteger(0) == 1 && tuple.getInteger(1) == 2; + } +} +``` + +Now suppose you had these tuples with fields ["a", "b", "c"]: + +``` +[1, 2, 3] +[2, 1, 1] +[2, 3, 4] +``` + +If you ran this code: + +```java +mystream.each(new Fields("b", "a"), new MyFilter()) +``` + +The resulting tuples would be: + +``` +[2, 1, 1] +``` + +### partitionAggregate + +partitionAggregate runs a function on each partition of a batch of tuples. Unlike functions, the tuples emitted by partitionAggregate replace the input tuples given to it. Consider this example: + +```java +mystream.partitionAggregate(new Fields("b"), new Sum(), new Fields("sum")) +``` + +Suppose the input stream contained fields ["a", "b"] and the following partitions of tuples: + +``` +Partition 0: +["a", 1] +["b", 2] + +Partition 1: +["a", 3] +["c", 8] + +Partition 2: +["e", 1] +["d", 9] +["d", 10] +``` + +Then the output stream of that code would contain these tuples with one field called "sum": + +``` +Partition 0: +[3] + +Partition 1: +[11] + +Partition 2: +[20] +``` + +There are three different interfaces for defining aggregators: CombinerAggregator, ReducerAggregator, and Aggregator. + +Here's the interface for CombinerAggregator: + +```java +public interface CombinerAggregator<T> extends Serializable { + T init(TridentTuple tuple); + T combine(T val1, T val2); + T zero(); +} +``` + +A CombinerAggregator returns a single tuple with a single field as output. CombinerAggregators run the init function on each input tuple and use the combine function to combine values until there's only one value left. If there's no tuples in the partition, the CombinerAggregator emits the output of the zero function. For example, here's the implementation of Count: + +```java +public class Count implements CombinerAggregator<Long> { + public Long init(TridentTuple tuple) { + return 1L; + } + + public Long combine(Long val1, Long val2) { + return val1 + val2; + } + + public Long zero() { + return 0L; + } +} +``` + +The benefits of CombinerAggregators are seen when you use the with the aggregate method instead of partitionAggregate. In that case, Trident automatically optimizes the computation by doing partial aggregations before transferring tuples over the network. + +A ReducerAggregator has the following interface: + +```java +public interface ReducerAggregator<T> extends Serializable { + T init(); + T reduce(T curr, TridentTuple tuple); +} +``` + +A ReducerAggregator produces an initial value with init, and then it iterates on that value for each input tuple to produce a single tuple with a single value as output. For example, here's how you would define Count as a ReducerAggregator: + +```java +public class Count implements ReducerAggregator<Long> { + public Long init() { + return 0L; + } + + public Long reduce(Long curr, TridentTuple tuple) { + return curr + 1; + } +} +``` + +ReducerAggregator can also be used with persistentAggregate, as you'll see later. + +The most general interface for performing aggregations is Aggregator, which looks like this: + +```java +public interface Aggregator<T> extends Operation { + T init(Object batchId, TridentCollector collector); + void aggregate(T state, TridentTuple tuple, TridentCollector collector); + void complete(T state, TridentCollector collector); +} +``` + +Aggregators can emit any number of tuples with any number of fields. They can emit tuples at any point during execution. Aggregators execute in the following way: + +1. The init method is called before processing the batch. The return value of init is an Object that will represent the state of the aggregation and will be passed into the aggregate and complete methods. +2. The aggregate method is called for each input tuple in the batch partition. This method can update the state and optionally emit tuples. +3. The complete method is called when all tuples for the batch partition have been processed by aggregate. + +Here's how you would implement Count as an Aggregator: + +```java +public class CountAgg extends BaseAggregator<CountState> { + static class CountState { + long count = 0; + } + + public CountState init(Object batchId, TridentCollector collector) { + return new CountState(); + } + + public void aggregate(CountState state, TridentTuple tuple, TridentCollector collector) { + state.count+=1; + } + + public void complete(CountState state, TridentCollector collector) { + collector.emit(new Values(state.count)); + } +} +``` + +Sometimes you want to execute multiple aggregators at the same time. This is called chaining and can be accomplished like this: + +```java +mystream.chainedAgg() + .partitionAggregate(new Count(), new Fields("count")) + .partitionAggregate(new Fields("b"), new Sum(), new Fields("sum")) + .chainEnd() +``` + +This code will run the Count and Sum aggregators on each partition. The output will contain a single tuple with the fields ["count", "sum"]. + +### stateQuery and partitionPersist + +stateQuery and partitionPersist query and update sources of state, respectively. You can read about how to use them on [Trident state doc](Trident-state.html). + +### projection + +The projection method on Stream keeps only the fields specified in the operation. If you had a Stream with fields ["a", "b", "c", "d"] and you ran this code: + +```java +mystream.project(new Fields("b", "d")) +``` + +The output stream would contain only the fields ["b", "d"]. + + +## Repartitioning operations + +Repartitioning operations run a function to change how the tuples are partitioned across tasks. The number of partitions can also change as a result of repartitioning (for example, if the parallelism hint is greater after repartioning). Repartitioning requires network transfer. Here are the repartitioning functions: + +1. shuffle: Use random round robin algorithm to evenly redistribute tuples across all target partitions +2. broadcast: Every tuple is replicated to all target partitions. This can useful during DRPC â for example, if you need to do a stateQuery on every partition of data. +3. partitionBy: partitionBy takes in a set of fields and does semantic partitioning based on that set of fields. The fields are hashed and modded by the number of target partitions to select the target partition. partitionBy guarantees that the same set of fields always goes to the same target partition. +4. global: All tuples are sent to the same partition. The same partition is chosen for all batches in the stream. +5. batchGlobal: All tuples in the batch are sent to the same partition. Different batches in the stream may go to different partitions. +6. partition: This method takes in a custom partitioning function that implements backtype.storm.grouping.CustomStreamGrouping + +## Aggregation operations + +Trident has aggregate and persistentAggregate methods for doing aggregations on Streams. aggregate is run on each batch of the stream in isolation, while persistentAggregate will aggregation on all tuples across all batches in the stream and store the result in a source of state. + +Running aggregate on a Stream does a global aggregation. When you use a ReducerAggregator or an Aggregator, the stream is first repartitioned into a single partition, and then the aggregation function is run on that partition. When you use a CombinerAggregator, on the other hand, first Trident will compute partial aggregations of each partition, then repartition to a single partition, and then finish the aggregation after the network transfer. CombinerAggregator's are far more efficient and should be used when possible. + +Here's an example of using aggregate to get a global count for a batch: + +```java +mystream.aggregate(new Count(), new Fields("count")) +``` + +Like partitionAggregate, aggregators for aggregate can be chained. However, if you chain a CombinerAggregator with a non-CombinerAggregator, Trident is unable to do the partial aggregation optimization. + +You can read more about how to use persistentAggregate in the [Trident state doc](https://github.com/apache/incubator-storm/wiki/Trident-state). + +## Operations on grouped streams + +The groupBy operation repartitions the stream by doing a partitionBy on the specified fields, and then within each partition groups tuples together whose group fields are equal. For example, here's an illustration of a groupBy operation: + + + +If you run aggregators on a grouped stream, the aggregation will be run within each group instead of against the whole batch. persistentAggregate can also be run on a GroupedStream, in which case the results will be stored in a [MapState](https://github.com/apache/incubator-storm/blob/master/storm-core/src/jvm/storm/trident/state/map/MapState.java) with the key being the grouping fields. You can read more about persistentAggregate in the [Trident state doc](Trident-state.html). + +Like regular streams, aggregators on grouped streams can be chained. + +## Merges and joins + +The last part of the API is combining different streams together. The simplest way to combine streams is to merge them into one stream. You can do that with the TridentTopology#merge method, like so: + +```java +topology.merge(stream1, stream2, stream3); +``` + +Trident will name the output fields of the new, merged stream as the output fields of the first stream. + +Another way to combine streams is with a join. Now, a standard join, like the kind from SQL, require finite input. So they don't make sense with infinite streams. Joins in Trident only apply within each small batch that comes off of the spout. + +Here's an example join between a stream containing fields ["key", "val1", "val2"] and another stream containing ["x", "val1"]: + +```java +topology.join(stream1, new Fields("key"), stream2, new Fields("x"), new Fields("key", "a", "b", "c")); +``` + +This joins stream1 and stream2 together using "key" and "x" as the join fields for each respective stream. Then, Trident requires that all the output fields of the new stream be named, since the input streams could have overlapping field names. The tuples emitted from the join will contain: + +1. First, the list of join fields. In this case, "key" corresponds to "key" from stream1 and "x" from stream2. +2. Next, a list of all non-join fields from all streams, in order of how the streams were passed to the join method. In this case, "a" and "b" correspond to "val1" and "val2" from stream1, and "c" corresponds to "val1" from stream2. + +When a join happens between streams originating from different spouts, those spouts will be synchronized with how they emit batches. That is, a batch of processing will include tuples from each spout. + +You might be wondering â how do you do something like a "windowed join", where tuples from one side of the join are joined against the last hour of tuples from the other side of the join. + +To do this, you would make use of partitionPersist and stateQuery. The last hour of tuples from one side of the join would be stored and rotated in a source of state, keyed by the join field. Then the stateQuery would do lookups by the join field to perform the "join". http://git-wip-us.apache.org/repos/asf/storm/blob/d491c3ff/docs/Trident-spouts.md ---------------------------------------------------------------------- diff --git a/docs/Trident-spouts.md b/docs/Trident-spouts.md new file mode 100644 index 0000000..92330a7 --- /dev/null +++ b/docs/Trident-spouts.md @@ -0,0 +1,42 @@ +--- +layout: documentation +--- +# Trident spouts + +Like in the vanilla Storm API, spouts are the source of streams in a Trident topology. On top of the vanilla Storm spouts, Trident exposes additional APIs for more sophisticated spouts. + +There is an inextricable link between how you source your data streams and how you update state (e.g. databases) based on those data streams. See [Trident state doc](Trident-state.html) for an explanation of this â understanding this link is imperative for understanding the spout options available. + +Regular Storm spouts will be non-transactional spouts in a Trident topology. To use a regular Storm IRichSpout, create the stream like this in a TridentTopology: + +```java +TridentTopology topology = new TridentTopology(); +topology.newStream("myspoutid", new MyRichSpout()); +``` + +All spouts in a Trident topology are required to be given a unique identifier for the stream â this identifier must be unique across all topologies run on the cluster. Trident will use this identifier to store metadata about what the spout has consumed in Zookeeper, including the txid and any metadata associated with the spout. + +You can configure the Zookeeper storage of spout metadata via the following configuration options: + +1. `transactional.zookeeper.servers`: A list of Zookeeper hostnames +2. `transactional.zookeeper.port`: The port of the Zookeeper cluster +3. `transactional.zookeeper.root`: The root dir in Zookeeper where metadata is stored. Metadata will be stored at the path <root path>/<spout id> + +## Pipelining + +By default, Trident processes a single batch at a time, waiting for the batch to succeed or fail before trying another batch. You can get significantly higher throughput â and lower latency of processing of each batch â by pipelining the batches. You configure the maximum amount of batches to be processed simultaneously with the "topology.max.spout.pending" property. + +Even while processing multiple batches simultaneously, Trident will order any state updates taking place in the topology among batches. For example, suppose you're doing a global count aggregation into a database. The idea is that while you're updating the count in the database for batch 1, you can still be computing the partial counts for batches 2 through 10. Trident won't move on to the state updates for batch 2 until the state updates for batch 1 have succeeded. This is essential for achieving exactly-once processing semantics, as outline in [Trident state doc](Trident-state.html). + +## Trident spout types + +Here are the following spout APIs available: + +1. [ITridentSpout](https://github.com/apache/incubator-storm/blob/master/storm-core/src/jvm/storm/trident/spout/ITridentSpout.java): The most general API that can support transactional or opaque transactional semantics. Generally you'll use one of the partitioned flavors of this API rather than this one directly. +2. [IBatchSpout](https://github.com/apache/incubator-storm/blob/master/storm-core/src/jvm/storm/trident/spout/IBatchSpout.java): A non-transactional spout that emits batches of tuples at a time +3. [IPartitionedTridentSpout](https://github.com/apache/incubator-storm/blob/master/storm-core/src/jvm/storm/trident/spout/IPartitionedTridentSpout.java): A transactional spout that reads from a partitioned data source (like a cluster of Kafka servers) +4. [IOpaquePartitionedTridentSpout](https://github.com/apache/incubator-storm/blob/master/storm-core/src/jvm/storm/trident/spout/IOpaquePartitionedTridentSpout.java): An opaque transactional spout that reads from a partitioned data source + +And, like mentioned in the beginning of this tutorial, you can use regular IRichSpout's as well. + + http://git-wip-us.apache.org/repos/asf/storm/blob/d491c3ff/docs/Trident-state.md ---------------------------------------------------------------------- diff --git a/docs/Trident-state.md b/docs/Trident-state.md new file mode 100644 index 0000000..2ace8c8 --- /dev/null +++ b/docs/Trident-state.md @@ -0,0 +1,330 @@ +--- +layout: documentation +--- +# State in Trident + +Trident has first-class abstractions for reading from and writing to stateful sources. The state can either be internal to the topology â e.g., kept in-memory and backed by HDFS â or externally stored in a database like Memcached or Cassandra. There's no difference in the Trident API for either case. + +Trident manages state in a fault-tolerant way so that state updates are idempotent in the face of retries and failures. This lets you reason about Trident topologies as if each message were processed exactly-once. + +There's various levels of fault-tolerance possible when doing state updates. Before getting to those, let's look at an example that illustrates the tricks necessary to achieve exactly-once semantics. Suppose that you're doing a count aggregation of your stream and want to store the running count in a database. Now suppose you store in the database a single value representing the count, and every time you process a new tuple you increment the count. + +When failures occur, tuples will be replayed. This brings up a problem when doing state updates (or anything with side effects) â you have no idea if you've ever successfully updated the state based on this tuple before. Perhaps you never processed the tuple before, in which case you should increment the count. Perhaps you've processed the tuple and successfully incremented the count, but the tuple failed processing in another step. In this case, you should not increment the count. Or perhaps you saw the tuple before but got an error when updating the database. In this case, you *should* update the database. + +By just storing the count in the database, you have no idea whether or not this tuple has been processed before. So you need more information in order to make the right decision. Trident provides the following semantics which are sufficient for achieving exactly-once processing semantics: + +1. Tuples are processed as small batches (see [the tutorial](Trident-tutorial.html)) +2. Each batch of tuples is given a unique id called the "transaction id" (txid). If the batch is replayed, it is given the exact same txid. +3. State updates are ordered among batches. That is, the state updates for batch 3 won't be applied until the state updates for batch 2 have succeeded. + +With these primitives, your State implementation can detect whether or not the batch of tuples has been processed before and take the appropriate action to update the state in a consistent way. The action you take depends on the exact semantics provided by your input spouts as to what's in each batch. There's three kinds of spouts possible with respect to fault-tolerance: "non-transactional", "transactional", and "opaque transactional". Likewise, there's three kinds of state possible with respect to fault-tolerance: "non-transactional", "transactional", and "opaque transactional". Let's take a look at each spout type and see what kind of fault-tolerance you can achieve with each. + +## Transactional spouts + +Remember, Trident processes tuples as small batches with each batch being given a unique transaction id. The properties of spouts vary according to the guarantees they can provide as to what's in each batch. A transactional spout has the following properties: + +1. Batches for a given txid are always the same. Replays of batches for a txid will exact same set of tuples as the first time that batch was emitted for that txid. +2. There's no overlap between batches of tuples (tuples are in one batch or another, never multiple). +3. Every tuple is in a batch (no tuples are skipped) + +This is a pretty easy type of spout to understand, the stream is divided into fixed batches that never change. storm-contrib has [an implementation of a transactional spout](https://github.com/nathanmarz/storm-contrib/blob/{{page.version}}/storm-kafka/src/jvm/storm/kafka/trident/TransactionalTridentKafkaSpout.java) for Kafka. + +You might be wondering â why wouldn't you just always use a transactional spout? They're simple and easy to understand. One reason you might not use one is because they're not necessarily very fault-tolerant. For example, the way TransactionalTridentKafkaSpout works is the batch for a txid will contain tuples from all the Kafka partitions for a topic. Once a batch has been emitted, any time that batch is re-emitted in the future the exact same set of tuples must be emitted to meet the semantics of transactional spouts. Now suppose a batch is emitted from TransactionalTridentKafkaSpout, the batch fails to process, and at the same time one of the Kafka nodes goes down. You're now incapable of replaying the same batch as you did before (since the node is down and some partitions for the topic are not unavailable), and processing will halt. + +This is why "opaque transactional" spouts exist â they are fault-tolerant to losing source nodes while still allowing you to achieve exactly-once processing semantics. We'll cover those spouts in the next section though. + +(One side note â once Kafka supports replication, it will be possible to have transactional spouts that are fault-tolerant to node failure, but that feature does not exist yet.) + +Before we get to "opaque transactional" spouts, let's look at how you would design a State implementation that has exactly-once semantics for transactional spouts. This State type is called a "transactional state" and takes advantage of the fact that any given txid is always associated with the exact same set of tuples. + +Suppose your topology computes word count and you want to store the word counts in a key/value database. The key will be the word, and the value will contain the count. You've already seen that storing just the count as the value isn't sufficient to know whether you've processed a batch of tuples before. Instead, what you can do is store the transaction id with the count in the database as an atomic value. Then, when updating the count, you can just compare the transaction id in the database with the transaction id for the current batch. If they're the same, you skip the update â because of the strong ordering, you know for sure that the value in the database incorporates the current batch. If they're different, you increment the count. This logic works because the batch for a txid never changes, and Trident ensures that state updates are ordered among batches. + +Consider this example of why it works. Suppose you are processing txid 3 which consists of the following batch of tuples: + +``` +["man"] +["man"] +["dog"] +``` + +Suppose the database currently holds the following key/value pairs: + +``` +man => [count=3, txid=1] +dog => [count=4, txid=3] +apple => [count=10, txid=2] +``` + +The txid associated with "man" is txid 1. Since the current txid is 3, you know for sure that this batch of tuples is not represented in that count. So you can go ahead and increment the count by 2 and update the txid. On the other hand, the txid for "dog" is the same as the current txid. So you know for sure that the increment from the current batch is already represented in the database for the "dog" key. So you can skip the update. After completing updates, the database looks like this: + +``` +man => [count=5, txid=3] +dog => [count=4, txid=3] +apple => [count=10, txid=2] +``` + +Let's now look at opaque transactional spouts and how to design states for that type of spout. + +## Opaque transactional spouts + +As described before, an opaque transactional spout cannot guarantee that the batch of tuples for a txid remains constant. An opaque transactional spout has the following property: + +1. Every tuple is *successfully* processed in exactly one batch. However, it's possible for a tuple to fail to process in one batch and then succeed to process in a later batch. + +[OpaqueTridentKafkaSpout](https://github.com/nathanmarz/storm-contrib/blob/{{page.version}}/storm-kafka/src/jvm/storm/kafka/trident/OpaqueTridentKafkaSpout.java) is a spout that has this property and is fault-tolerant to losing Kafka nodes. Whenever it's time for OpaqueTridentKafkaSpout to emit a batch, it emits tuples starting from where the last batch finished emitting. This ensures that no tuple is ever skipped or successfully processed by multiple batches. + +With opaque transactional spouts, it's no longer possible to use the trick of skipping state updates if the transaction id in the database is the same as the transaction id for the current batch. This is because the batch may have changed between state updates. + +What you can do is store more state in the database. Rather than store a value and transaction id in the database, you instead store a value, transaction id, and the previous value in the database. Let's again use the example of storing a count in the database. Suppose the partial count for your batch is "2" and it's time to apply a state update. Suppose the value in the database looks like this: + +``` +{ value = 4, + prevValue = 1, + txid = 2 +} +``` + +Suppose your current txid is 3, different than what's in the database. In this case, you set "prevValue" equal to "value", increment "value" by your partial count, and update the txid. The new database value will look like this: + +``` +{ value = 6, + prevValue = 4, + txid = 3 +} +``` + +Now suppose your current txid is 2, equal to what's in the database. Now you know that the "value" in the database contains an update from a previous batch for your current txid, but that batch may have been different so you have to ignore it. What you do in this case is increment "prevValue" by your partial count to compute the new "value". You then set the value in the database to this: + +``` +{ value = 3, + prevValue = 1, + txid = 2 +} +``` + +This works because of the strong ordering of batches provided by Trident. Once Trident moves onto a new batch for state updates, it will never go back to a previous batch. And since opaque transactional spouts guarantee no overlap between batches â that each tuple is successfully processed by one batch â you can safely update based on the previous value. + +## Non-transactional spouts + +Non-transactional spouts don't provide any guarantees about what's in each batch. So it might have at-most-once processing, in which case tuples are not retried after failed batches. Or it might have at-least-once processing, where tuples can be processed successfully by multiple batches. There's no way to achieve exactly-once semantics for this kind of spout. + +## Summary of spout and state types + +This diagram shows which combinations of spouts / states enable exactly-once messaging semantics: + + + +Opaque transactional states have the strongest fault-tolerance, but this comes at the cost of needing to store the txid and two values in the database. Transactional states require less state in the database, but only work with transactional spouts. Finally, non-transactional states require the least state in the database but cannot achieve exactly-once semantics. + +The state and spout types you choose are a tradeoff between fault-tolerance and storage costs, and ultimately your application requirements will determine which combination is right for you. + +## State APIs + +You've seen the intricacies of what it takes to achieve exactly-once semantics. The nice thing about Trident is that it internalizes all the fault-tolerance logic within the State â as a user you don't have to deal with comparing txids, storing multiple values in the database, or anything like that. You can write code like this: + +```java +TridentTopology topology = new TridentTopology(); +TridentState wordCounts = + topology.newStream("spout1", spout) + .each(new Fields("sentence"), new Split(), new Fields("word")) + .groupBy(new Fields("word")) + .persistentAggregate(MemcachedState.opaque(serverLocations), new Count(), new Fields("count")) + .parallelismHint(6); +``` + +All the logic necessary to manage opaque transactional state logic is internalized in the MemcachedState.opaque call. Additionally, updates are automatically batched to minimize roundtrips to the database. + +The base State interface just has two methods: + +```java +public interface State { + void beginCommit(Long txid); // can be null for things like partitionPersist occurring off a DRPC stream + void commit(Long txid); +} +``` + +You're told when a state update is beginning, when a state update is ending, and you're given the txid in each case. Trident assumes nothing about how your state works, what kind of methods there are to update it, and what kind of methods there are to read from it. + +Suppose you have a home-grown database that contains user location information and you want to be able to access it from Trident. Your State implementation would have methods for getting and setting user information: + +```java +public class LocationDB implements State { + public void beginCommit(Long txid) { + } + + public void commit(Long txid) { + } + + public void setLocation(long userId, String location) { + // code to access database and set location + } + + public String getLocation(long userId) { + // code to get location from database + } +} +``` + +You then provide Trident a StateFactory that can create instances of your State object within Trident tasks. The StateFactory for your LocationDB might look something like this: + +```java +public class LocationDBFactory implements StateFactory { + public State makeState(Map conf, int partitionIndex, int numPartitions) { + return new LocationDB(); + } +} +``` + +Trident provides the QueryFunction interface for writing Trident operations that query a source of state, and the StateUpdater interface for writing Trident operations that update a source of state. For example, let's write an operation "QueryLocation" that queries the LocationDB for the locations of users. Let's start off with how you would use it in a topology. Let's say this topology consumes an input stream of userids: + +```java +TridentTopology topology = new TridentTopology(); +TridentState locations = topology.newStaticState(new LocationDBFactory()); +topology.newStream("myspout", spout) + .stateQuery(locations, new Fields("userid"), new QueryLocation(), new Fields("location")) +``` + +Now let's take a look at what the implementation of QueryLocation would look like: + +```java +public class QueryLocation extends BaseQueryFunction<LocationDB, String> { + public List<String> batchRetrieve(LocationDB state, List<TridentTuple> inputs) { + List<String> ret = new ArrayList(); + for(TridentTuple input: inputs) { + ret.add(state.getLocation(input.getLong(0))); + } + return ret; + } + + public void execute(TridentTuple tuple, String location, TridentCollector collector) { + collector.emit(new Values(location)); + } +} +``` + +QueryFunction's execute in two steps. First, Trident collects a batch of reads together and passes them to batchRetrieve. In this case, batchRetrieve will receive multiple user ids. batchRetrieve is expected to return a list of results that's the same size as the list of input tuples. The first element of the result list corresponds to the result for the first input tuple, the second is the result for the second input tuple, and so on. + +You can see that this code doesn't take advantage of the batching that Trident does, since it just queries the LocationDB one at a time. So a better way to write the LocationDB would be like this: + +```java +public class LocationDB implements State { + public void beginCommit(Long txid) { + } + + public void commit(Long txid) { + } + + public void setLocationsBulk(List<Long> userIds, List<String> locations) { + // set locations in bulk + } + + public List<String> bulkGetLocations(List<Long> userIds) { + // get locations in bulk + } +} +``` + +Then, you can write the QueryLocation function like this: + +```java +public class QueryLocation extends BaseQueryFunction<LocationDB, String> { + public List<String> batchRetrieve(LocationDB state, List<TridentTuple> inputs) { + List<Long> userIds = new ArrayList<Long>(); + for(TridentTuple input: inputs) { + userIds.add(input.getLong(0)); + } + return state.bulkGetLocations(userIds); + } + + public void execute(TridentTuple tuple, String location, TridentCollector collector) { + collector.emit(new Values(location)); + } +} +``` + +This code will be much more efficient by reducing roundtrips to the database. + +To update state, you make use of the StateUpdater interface. Here's a StateUpdater that updates a LocationDB with new location information: + +```java +public class LocationUpdater extends BaseStateUpdater<LocationDB> { + public void updateState(LocationDB state, List<TridentTuple> tuples, TridentCollector collector) { + List<Long> ids = new ArrayList<Long>(); + List<String> locations = new ArrayList<String>(); + for(TridentTuple t: tuples) { + ids.add(t.getLong(0)); + locations.add(t.getString(1)); + } + state.setLocationsBulk(ids, locations); + } +} +``` + +Here's how you would use this operation in a Trident topology: + +```java +TridentTopology topology = new TridentTopology(); +TridentState locations = + topology.newStream("locations", locationsSpout) + .partitionPersist(new LocationDBFactory(), new Fields("userid", "location"), new LocationUpdater()) +``` + +The partitionPersist operation updates a source of state. The StateUpdater receives the State and a batch of tuples with updates to that State. This code just grabs the userids and locations from the input tuples and does a bulk set into the State. + +partitionPersist returns a TridentState object representing the location db being updated by the Trident topology. You could then use this state in stateQuery operations elsewhere in the topology. + +You can also see that StateUpdaters are given a TridentCollector. Tuples emitted to this collector go to the "new values stream". In this case, there's nothing interesting to emit to that stream, but if you were doing something like updating counts in a database, you could emit the updated counts to that stream. You can then get access to the new values stream for further processing via the TridentState#newValuesStream method. + +## persistentAggregate + +Trident has another method for updating States called persistentAggregate. You've seen this used in the streaming word count example, shown again below: + +```java +TridentTopology topology = new TridentTopology(); +TridentState wordCounts = + topology.newStream("spout1", spout) + .each(new Fields("sentence"), new Split(), new Fields("word")) + .groupBy(new Fields("word")) + .persistentAggregate(new MemoryMapState.Factory(), new Count(), new Fields("count")) +``` + +persistentAggregate is an additional abstraction built on top of partitionPersist that knows how to take a Trident aggregator and use it to apply updates to the source of state. In this case, since this is a grouped stream, Trident expects the state you provide to implement the "MapState" interface. The grouping fields will be the keys in the state, and the aggregation result will be the values in the state. The "MapState" interface looks like this: + +```java +public interface MapState<T> extends State { + List<T> multiGet(List<List<Object>> keys); + List<T> multiUpdate(List<List<Object>> keys, List<ValueUpdater> updaters); + void multiPut(List<List<Object>> keys, List<T> vals); +} +``` + +When you do aggregations on non-grouped streams (a global aggregation), Trident expects your State object to implement the "Snapshottable" interface: + +```java +public interface Snapshottable<T> extends State { + T get(); + T update(ValueUpdater updater); + void set(T o); +} +``` + +[MemoryMapState](https://github.com/apache/incubator-storm/blob/{{page.version}}/storm-core/src/jvm/storm/trident/testing/MemoryMapState.java) and [MemcachedState](https://github.com/nathanmarz/trident-memcached/blob/master/src/jvm/trident/memcached/MemcachedState.java) each implement both of these interfaces. + +## Implementing Map States + +Trident makes it easy to implement MapState's, doing almost all the work for you. The OpaqueMap, TransactionalMap, and NonTransactionalMap classes implement all the logic for doing the respective fault-tolerance logic. You simply provide these classes with an IBackingMap implementation that knows how to do multiGets and multiPuts of the respective key/values. IBackingMap looks like this: + +```java +public interface IBackingMap<T> { + List<T> multiGet(List<List<Object>> keys); + void multiPut(List<List<Object>> keys, List<T> vals); +} +``` + +OpaqueMap's will call multiPut with [OpaqueValue](https://github.com/apache/incubator-storm/blob/{{page.version}}/storm-core/src/jvm/storm/trident/state/OpaqueValue.java)'s for the vals, TransactionalMap's will give [TransactionalValue](https://github.com/apache/incubator-storm/blob/{{page.version}}/storm-core/src/jvm/storm/trident/state/TransactionalValue.java)'s for the vals, and NonTransactionalMaps will just pass the objects from the topology through. + +Trident also provides the [CachedMap](https://github.com/apache/incubator-storm/blob/{{page.version}}/storm-core/src/jvm/storm/trident/state/map/CachedMap.java) class to do automatic LRU caching of map key/vals. + +Finally, Trident provides the [SnapshottableMap](https://github.com/apache/incubator-storm/blob/{{page.version}}/storm-core/src/jvm/storm/trident/state/map/SnapshottableMap.java) class that turns a MapState into a Snapshottable object, by storing global aggregations into a fixed key. + +Take a look at the implementation of [MemcachedState](https://github.com/nathanmarz/trident-memcached/blob/master/src/jvm/trident/memcached/MemcachedState.java) to see how all these utilities can be put together to make a high performance MapState implementation. MemcachedState allows you to choose between opaque transactional, transactional, and non-transactional semantics.
