Achieving exactly-once processing with Flink and InfluxDB

Leroux Romain
Lumen Engineering Blog
7 min readJun 5, 2019

We have talked a lot about data on the Streamroot tech blog, and for good reason. Powering millions of video sessions online every day, we have a lot of material to work with. Delivering video content — especially with hybrid CDN and peer-to-peer systems — is a highly complex undertaking that requires us to continuously collect and process feedback events to gauge its performance and make sure everything is working properly.

As we discussed in previous posts, Streamroot uses its data pipeline to AB test new features, to understand the effects of external factors on our delivery solution, and to improve the overall quality of our products. It is also used to expose certain traffic data to our customers via our graphic user interface, which is what I’d like to talk about today.

Put simply, we send feedback metrics as json payloads to a Kafka broker from where various consumers process those metrics for different purposes. One of our critical consumers is responsible for feeding our customer-facing time series database InfluxDB. This database is accessed by our customer dashboard, which displays a variety of peer-to-peer delivery and quality of service data for the broadcaster’s streams, platforms, ISPs, etc. In this post I’d like to focus on this critical consumer whose job is to pre-aggregate time series data (hence its name, the aggregator) so that InfluxDB isn’t overloaded and can respond smoothly.

This aggregator is actually a Flink job that:

  • Reads data from Kafka
  • Groups data by some key of interest (for instance a stream id)
  • Aggregates metrics for each key (for instance counting unique users on a given stream)
  • Does all that on time windows of 5 minutes
  • Sends the results to InfluxDB

Flink has really good built-in windowing and aggregation support so most steps weren’t too difficult to implement. However, the last step — that is, sending results to InfluxDB — was tricky to get right as many things can go wrong. For instance, InfluxDB can be down, a Flink machine can fail, etc. And since the resulting data will be exposed to our customers we needed to have strong guarantees such as exactly-once processing and no data loss.

To ensure that stateful computations are resilient, Flink provides a checkpoint based-mechanism. At regular intervals, it snapshots the states of the stateful components of a given job and saves it to an external backend (for instance files on a distributed storage such as HDFS). As an end user, what we have to do is essentially implement interfaces for our custom components and Flink will do the rest!

In this post we’ll implement a fully resilient InfluxDB sink (Flink’s abstraction for outputting data); we’ll start with a bare-bones implementation and improve it gradually. All of the code is available on Github and the tests can be run locally through a docker-compose setup that is also provided (it will spin up Flink and InfluxDB).

Basic implementation — InfluxSinkV1

First of all, let’s write a simple implementation of a sink writing data points to InfluxDB.

A sink class must implement SinkFunction<IN>; however, here we choose to extend RichSinkFunction<IN>. The reason for this is that the rich version provides a public void open(...) method that can be used to initialize any kind of complex/non-serializable state. In this case, the transient InfluxDB client is not Serializable which is why we have to create it through this open(...) method.

The main sink method invoke(...) is quite simple: it takes a Point and uses the influx client to send it.

Let’s check that everything is working correctly with a quick unit test:

Assuming that you have docker-compose running, as mentioned in the introduction, you can run this test with:

Everything looks good! However, there’s an issue. Writing data points one by one, every time that invoke(...) is called is quite inefficient for InfluxDB. So let's see what we can do about that.

Adding batching support — InfluxSinkV2

It turns out that the influx client has built-in support for batching! To use it all we need to do is to call enableBatch(...) as follows:

As for the unit test, we need to change it a bit to take into account the batching duration:

You can test it with:

But is it really that simple to handle batching properly? What happens if a failure occurs between two batches? It’s very likely that data points being buffered will get lost as Flink has no idea that we are buffering them.

So let’s see how we can handle a stateful sink properly with Flink.

Fully resilient implementation — InfluxSinkV3

To achieve resiliency we need to have the following mechanisms in place:

  1. Explicitly handle buffered points (leverage Flink’s checkpoints)
  2. Control batches of data points sent to InfluxDB
  3. Retry in case of failure (stop processing until recovery succeeds)

Explicitly handle buffered points

Flink’s documentation mentions managed operator states for handling this situation. This mechanism requires us to implement CheckpointedFunction to leverage Flink's checkpoints and thus be able to snapshot/recover the data points that are being buffered and batch sent.

Here are the two new methods that we will have to implement:

So we have to buffer points within our sink implementation. Let’s do that with a simple List<Point>:

In this new version our sink’s main method invoke(...) is now simply buffering points up to batchSize:

Let’s implement snapshotState(...) and initializeState(...) to respectively backup and restore bufferedPoints through checkpointedState:

Control batches of data points sent to InfluxDB

Since we stopped relying on the influx client’s built-in batching feature, we will have to roll our own. Luckily we can leverage OkHttpClient, on top of which influx client is built, to achieve batching.

To this end, we’ll start by defining a custom InfluxBatchService as follows:

Now let’s come back to the mysterious batchWrite(bufferedPoints) that we skipped earlier in the new invoke(...) method and let's see how it uses our new InfluxBatchService:

It simply iterates through data points and call lineProtocol to format them to the InfluxDB protocol before batch sending them with a POST request.

There has been a lot of changes since InfluxSinkV1 already, but are we done? Not yet! With our new batching approach we're flushing data points only when our buffer is full. What if we buffer points that are the result of 10-minute windowed aggregations and that our buffer isn't full? Well, they won't be flushed until the next 10-minute window ... Not very nice.

So let’s add a final piece to our custom batching implementation: a time limit. The idea is that we want to flush our buffered point as soon as one of the two following conditions is met: either the batchSize is reached, or batchFreqMs has elapsed.

To achieve that we can make use of a SingleThreadScheduledExecutor that will regularly flush the buffered points. To make sure that everything is thread-safe we can simply add synchronized to all methods interacting with bufferedPoints.

Let’s not forget to clean up the scheduled executor when the sink is being stopped:

With this new mechanism, invoke(...) will also have to make use of flushPoints():

And now flushPoints() is responsible for triggering batchWrite(bufferedPoints) and clearing the buffer:

We finally have a complete batching mechanism in place! But as you can see from the last line in flushPoints(), we're not doing much when an exception occurs... This brings us to our last feature: proper error handling and retries.

Retry in case of failure

The last piece of this InfluxSinkV3 is the retry mechanism that will make use of the Failsafe library. What we want is to stop processing when an InfluxDB related error occurs, and have a retry policy to help recover the situation.

Let’s define a retry policy that will just keep on retrying to send the current batch of data at regular intervals:

Now we wrap our call to batchWrite(bufferedPoints) with a Failsafe.with(retryPolicy) call. It will handle IOException s and errors returned by InfluxDB by retrying infinitely.

Last but not least, let’s modify batchWrite(...) so that it can be used by Failsafe functions. Let's also keep track of the number of retries.

And at last we’re all done! For completeness sake here is the final version of InfluxSinkV3.

Testing resiliency

Let’s summarize all the features we implemented:

  • Back up and restore state with bufferedPoints / checkpointedState
  • Custom InfluxBatchService with configurable flushing of batchSize / batchFreqMs
  • Error handling and retries with Failsafe

Our features look good; now we’d like to verify them with some unit tests. Let’s at least check that our retry mechanism is working well.

For that we’ll use a library called NetCrusher that can simulate a network failure. Let's modify our test setup so as to proxy connections to our local InfluxDB through a TcpCrusher so that we can stop and restart TCP traffic at will.

Our new resiliency test looks like this:

Again assuming that you have docker-compose running, you can run this test with:

The test should pass and we can see in the output that it has retried many times before successfully sending the current batch:

Improving usability with a builder — InfluxSinkV4

We ended up with a lot of parameters for our InfluxSink which make it a bit cumbersome to use. A simple solution for that is to use a builder to set up all parameters in a clean and fluent fashion.

Let’s see it in action with a complete Flink job that will:

  • Read and parse integer data points from a network socket
  • Sum data over 10-second windows
  • Send the results to a local InfluxDB (running in docker)

Complete instructions to run this job can be found on Github.

Conclusion

It’s been quite a journey from the naive InfluxSinkV1 to the fully resilient InfluxSinkV4 but it was worth it! We’ve addressed efficiency issues and stateful computation pitfalls when working with Flink and InfluxDB:

  • Batch write data points to InfluxDB
  • Regularly back up buffered points with Flink’s checkpoints
  • Handle errors with a fine-tuned retry policy

These mechanisms and the resulting exactly-once processing guarantees have helped us build reliable and scalable data pipelines that give our customers valuable insights about their video traffic. I hope the local demo application highlighted it and that you can use this as a basis for your own Flink applications. Stay tuned for more insights on how we fine tune our data pipeline, and if you are interested in working on these types of challenges, drop us a line at jobs@streamroot.io.

--

--