blob: d84128eb40fe889ceb497845d45217943686fc43 [file] [log] [blame]
<!DOCTYPE html>
<html lang="en" dir=>
<head>
<meta name="generator" content="Hugo 0.92.2" />
<meta charset="UTF-8">
<meta name="viewport" content="width=device-width, initial-scale=1.0">
<meta name="description" content="Stream processing applications are often stateful, “remembering” information from processed events and using it to influence further event processing. In Flink, the remembered information, i.e., state, is stored locally in the configured state backend. To prevent data loss in case of failures, the state backend periodically persists a snapshot of its contents to a pre-configured durable storage. The RocksDB state backend (i.e., RocksDBStateBackend) is one of the three built-in state backends in Flink.">
<meta name="theme-color" content="#FFFFFF"><meta property="og:title" content="Using RocksDB State Backend in Apache Flink: When and How" />
<meta property="og:description" content="Stream processing applications are often stateful, “remembering” information from processed events and using it to influence further event processing. In Flink, the remembered information, i.e., state, is stored locally in the configured state backend. To prevent data loss in case of failures, the state backend periodically persists a snapshot of its contents to a pre-configured durable storage. The RocksDB state backend (i.e., RocksDBStateBackend) is one of the three built-in state backends in Flink." />
<meta property="og:type" content="article" />
<meta property="og:url" content="https://flink.apache.org/2021/01/18/using-rocksdb-state-backend-in-apache-flink-when-and-how/" /><meta property="article:section" content="posts" />
<meta property="article:published_time" content="2021-01-18T00:00:00+00:00" />
<meta property="article:modified_time" content="2021-01-18T00:00:00+00:00" />
<title>Using RocksDB State Backend in Apache Flink: When and How | Apache Flink</title>
<link rel="manifest" href="/manifest.json">
<link rel="icon" href="/favicon.png" type="image/x-icon">
<link rel="stylesheet" href="/book.min.e3b33391dbc1f4b2cc47778e2f4b86c744ded3ccc82fdfb6f08caf91d8607f9a.css" integrity="sha256-47MzkdvB9LLMR3eOL0uGx0Te08zIL9&#43;28Iyvkdhgf5o=">
<script defer src="/en.search.min.069123ef70eb5a313a1a8ba877092ebbe3e8ff1540b613f6c3c0bdd7a30c0733.js" integrity="sha256-BpEj73DrWjE6Gouodwkuu&#43;Po/xVAthP2w8C916MMBzM="></script>
<!--
Made with Book Theme
https://github.com/alex-shpak/hugo-book
-->
<link rel="stylesheet" type="text/css" href="/font-awesome/css/font-awesome.min.css">
<script src="/js/anchor.min.js"></script>
<script src="/js/flink.js"></script>
<link rel="canonical" href="https://flink.apache.org/2021/01/18/using-rocksdb-state-backend-in-apache-flink-when-and-how/">
<script>
var _paq = window._paq = window._paq || [];
_paq.push(['disableCookies']);
_paq.push(["setDomains", ["*.flink.apache.org","*.nightlies.apache.org/flink"]]);
_paq.push(['trackPageView']);
_paq.push(['enableLinkTracking']);
(function() {
var u="//analytics.apache.org/";
_paq.push(['setTrackerUrl', u+'matomo.php']);
_paq.push(['setSiteId', '1']);
var d=document, g=d.createElement('script'), s=d.getElementsByTagName('script')[0];
g.async=true; g.src=u+'matomo.js'; s.parentNode.insertBefore(g,s);
})();
</script>
</head>
<body dir=>
<input type="checkbox" class="hidden toggle" id="menu-control" />
<input type="checkbox" class="hidden toggle" id="toc-control" />
<main class="container flex">
<aside class="book-menu">
<nav>
<a id="logo" href="/">
<img width="70%" src="/flink-header-logo.svg">
</a>
<div class="book-search">
<input type="text" id="book-search-input" placeholder="Search" aria-label="Search" maxlength="64" data-hotkeys="s/" />
<div class="book-search-spinner hidden"></div>
<ul id="book-search-results"></ul>
</div>
<input type="checkbox" id="section-4117fb24454a2c30ee86e524839e77ec" class="toggle" />
<label for="section-4117fb24454a2c30ee86e524839e77ec" class="flex justify-between flink-menu-item">What is Apache Flink?<span></span>
</label>
<ul>
<li>
<label for="section-ffd5922da551e96e0481423fab94c463" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/what-is-flink/flink-architecture/" class="">Architecture</a>
</label>
</li>
<li>
<label for="section-fc28f08b67476edb77e00e03b6c7c2e0" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/what-is-flink/flink-applications/" class="">Applications</a>
</label>
</li>
<li>
<label for="section-612df33a02d5d4ee78d718abaab5b5b4" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/what-is-flink/flink-operations/" class="">Operations</a>
</label>
</li>
</ul>
<label for="section-f1ecec07350bd6810050d40158878749" class="flex justify-between flink-menu-item">
<a href="https://nightlies.apache.org/flink/flink-statefun-docs-stable/" style="color:black" class="">What is Stateful Functions? <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<label for="section-4113a4c3072cb35f6fd7a0d4e098ee70" class="flex justify-between flink-menu-item">
<a href="https://nightlies.apache.org/flink/flink-ml-docs-stable/" style="color:black" class="">What is Flink ML? <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<label for="section-b39c70259d0abbe2bf1d8d645425f84d" class="flex justify-between flink-menu-item">
<a href="https://nightlies.apache.org/flink/flink-kubernetes-operator-docs-stable/" style="color:black" class="">What is the Flink Kubernetes Operator? <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<label for="section-53e0b1afcb9ccaf779dc285aa272a014" class="flex justify-between flink-menu-item">
<a href="https://nightlies.apache.org/flink/flink-table-store-docs-stable/" style="color:black" class="">What is Flink Table Store? <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<label for="section-f4973f06a66f063045b4ebdacaf3127d" class="flex justify-between flink-menu-item">
<a href="/use-cases/" class="">Use Cases</a>
</label>
<label for="section-0f1863835376e859ac438ae9529daff2" class="flex justify-between flink-menu-item">
<a href="/powered-by/" class="">Powered By</a>
</label>
<br/>
<label for="section-f383f23a96a43d8d0cc66aeb0237e26a" class="flex justify-between flink-menu-item">
<a href="/downloads/" class="">Downloads</a>
</label>
<input type="checkbox" id="section-c727fab97b4d77e5b28ce8c448fb9000" class="toggle" />
<label for="section-c727fab97b4d77e5b28ce8c448fb9000" class="flex justify-between flink-menu-item">Getting Started<span></span>
</label>
<ul>
<li>
<label for="section-f45abaa99ab076108b9a5b94edbc6647" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-docs-stable/docs/try-flink/local_installation/" style="color:black" class="">With Flink <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-efe2166e9dce6f72e126dcc2396b4402" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-statefun-docs-stable/getting-started/project-setup.html" style="color:black" class="">With Flink Stateful Functions <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-7e268d0a469b1093bb33d71d093eb7b9" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-ml-docs-stable/docs/try-flink-ml/quick-start/" style="color:black" class="">With Flink ML <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-cc7147cd0441503127bfaf6f219d4fbb" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-kubernetes-operator-docs-stable/docs/try-flink-kubernetes-operator/quick-start/" style="color:black" class="">With Flink Kubernetes Operator <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-660ca694e416d8ca9176dda52a60d637" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-table-store-docs-stable/docs/try-table-store/quick-start/" style="color:black" class="">With Flink Table Store <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-75db0b47bf4ae9c247aadbba5fbd720d" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-docs-stable/docs/learn-flink/overview/" style="color:black" class="">Training Course <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
</ul>
<input type="checkbox" id="section-6318075fef29529089951a49d413d083" class="toggle" />
<label for="section-6318075fef29529089951a49d413d083" class="flex justify-between flink-menu-item">Documentation<span></span>
</label>
<ul>
<li>
<label for="section-9a8122d8912450484d1c25394ad40229" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-docs-stable/" style="color:black" class="">Flink 1.17 (stable) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-8b2fd3efb702be3783ba98d650707e3c" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-docs-master/" style="color:black" class="">Flink Master (snapshot) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-5317a079cddb964c59763c27607f43d9" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-statefun-docs-stable/" style="color:black" class="">Stateful Functions 3.2 (stable) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-25b72f108b7156e94d91b04853d8813a" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-statefun-docs-master" style="color:black" class="">Stateful Functions Master (snapshot) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-13a02f969904a2455a39ed90e287593f" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-ml-docs-stable/" style="color:black" class="">ML 2.2 (stable) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-6d895ec5ad127a29a6a9ce101328ccdf" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-ml-docs-master" style="color:black" class="">ML Master (snapshot) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-c83ad0caf34e364bf3729badd233a350" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-kubernetes-operator-docs-stable/" style="color:black" class="">Kubernetes Operator 1.4 (latest) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-a2c75d90005425982ba8f26ae0e160a3" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-kubernetes-operator-docs-main" style="color:black" class="">Kubernetes Operator Main (snapshot) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-07b85e4b2f61b1526bf202c64460abcd" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-table-store-docs-stable/" style="color:black" class="">Table Store 0.3 (stable) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
<li>
<label for="section-9b9a0032b1e858a34c125d828d1a0718" class="flex justify-between flink-menu-item flink-menu-child">
<a href="https://nightlies.apache.org/flink/flink-table-store-docs-master/" style="color:black" class="">Table Store Master (snapshot) <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
</li>
</ul>
<label for="section-63d6a565d79aa2895f70806a46021c07" class="flex justify-between flink-menu-item">
<a href="/getting-help/" class="">Getting Help</a>
</label>
<label for="section-1d5066022b83f4732dc80f4e9eaa069a" class="flex justify-between flink-menu-item">
<a href="https://flink-packages.org/" style="color:black" class="">flink-packages.org <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<br/>
<label for="section-7821b78a97db9e919426e86121a7be9c" class="flex justify-between flink-menu-item">
<a href="/community/" class="">Community & Project Info</a>
</label>
<label for="section-8c042831df4e371c4ef9375f1df06f35" class="flex justify-between flink-menu-item">
<a href="/roadmap/" class="">Roadmap</a>
</label>
<input type="checkbox" id="section-73117efde5302fddcb193307d582b588" class="toggle" />
<label for="section-73117efde5302fddcb193307d582b588" class="flex justify-between flink-menu-item">How to Contribute<span></span>
</label>
<ul>
<li>
<label for="section-6646b26b23a3e79b8de9c552ee76f6dd" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/overview/" class="">Overview</a>
</label>
</li>
<li>
<label for="section-e6ab9538b82cd5f94103b971adb7c1a9" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/contribute-code/" class="">Contribute Code</a>
</label>
</li>
<li>
<label for="section-1c09e1358485e82d9b3f5f689d4ced65" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/reviewing-prs/" class="">Review Pull Requests</a>
</label>
</li>
<li>
<label for="section-ed01e0defd235498fa3c9a2a0b3302fb" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/code-style-and-quality-preamble/" class="">Code Style and Quality Guide</a>
</label>
</li>
<li>
<label for="section-4e8d5e9924cf15f397711b0d82e15650" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/contribute-documentation/" class="">Contribute Documentation</a>
</label>
</li>
<li>
<label for="section-ddaa8307917e5ba7f60ba3316711e492" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/documentation-style-guide/" class="">Documentation Style Guide</a>
</label>
</li>
<li>
<label for="section-390a72c171cc82f180a308b95fc3aa72" class="flex justify-between flink-menu-item flink-menu-child">
<a href="/how-to-contribute/improve-website/" class="">Contribute to the Website</a>
</label>
</li>
</ul>
<label for="section-9d3ddfd487223d5a199ba301f25c88c6" class="flex justify-between flink-menu-item">
<a href="/security/" class="">Security</a>
</label>
<br/>
<label for="section-a07783f405300745807d39eacf150420" class="flex justify-between flink-menu-item">
<a href="/posts/" class="">Flink Blog</a>
</label>
<br/>
<hr class="menu-break">
<label for="section-f71a7070dbb7b669824a6441408ded70" class="flex justify-between flink-menu-item">
<a href="https://github.com/apache/flink" style="color:black" class="">Flink on GitHub <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<label for="section-2ccaaab8c67f3105bbf7df75faca8027" class="flex justify-between flink-menu-item">
<a href="https://twitter.com/apacheflink" style="color:black" class="">@ApacheFlink <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label>
<hr class="menu-break">
<table>
<tr>
<th colspan="2">
<label for="section-78c2028200542d78f8c1a8f6b4cbb36b" class="flex justify-between flink-menu-item">
<a href="https://www.apache.org/" style="color:black" class="">Apache Software Foundation <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label></th>
</tr>
<tr>
<td>
<label for="section-794df3791a8c800841516007427a2aa3" class="flex justify-between flink-menu-item">
<a href="https://www.apache.org/licenses/" style="color:black" class="">License <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label></td>
<td>
<label for="section-2fae32629d4ef4fc6341f1751b405e45" class="flex justify-between flink-menu-item">
<a href="https://www.apache.org/security/" style="color:black" class="">Security <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label></td>
</tr>
<tr>
<td>
<label for="section-0584e445d656b83b431227bb80ff0c30" class="flex justify-between flink-menu-item">
<a href="https://www.apache.org/foundation/sponsorship.html" style="color:black" class="">Donate <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label></td>
<td>
<label for="section-00d06796e489999226fb5bb27fe1b3b2" class="flex justify-between flink-menu-item">
<a href="https://www.apache.org/foundation/thanks.html" style="color:black" class="">Thanks <i class="link fa fa-external-link title" aria-hidden="true"></i></a>
</label></td>
</tr>
</table>
<hr class="menu-break">
<a href="/zh/" class="flex align-center">
<i class="fa fa-globe" aria-hidden="true"></i>&nbsp;&nbsp;
中文版
</a>
<script src="/js/track-search-terms.js"></script>
</nav>
<script>(function(){var a=document.querySelector("aside.book-menu nav");addEventListener("beforeunload",function(b){localStorage.setItem("menu.scrollTop",a.scrollTop)}),a.scrollTop=localStorage.getItem("menu.scrollTop")})()</script>
</aside>
<div class="book-page">
<header class="book-header">
<div class="flex align-center justify-between">
<label for="menu-control">
<img src="/svg/menu.svg" class="book-icon" alt="Menu" />
</label>
<strong>Using RocksDB State Backend in Apache Flink: When and How</strong>
<label for="toc-control">
<img src="/svg/toc.svg" class="book-icon" alt="Table of Contents" />
</label>
</div>
<aside class="hidden clearfix">
<nav id="TableOfContents"><h3>On This Page <button class="toc" onclick="collapseToc()"><i class="fa fa-compress" aria-hidden="true"></i></button></h3>
<ul>
<li><a href="#cluster-level">Cluster Level</a></li>
</ul>
<ul>
<li><a href="#state-location-in-rocksdb">State Location in RocksDB</a></li>
<li><a href="#troubleshooting-rocksdb">Troubleshooting RocksDB</a></li>
<li><a href="#tuning-rocksdb">Tuning RocksDB</a></li>
</ul>
</nav>
</aside>
</header>
<article class="markdown">
<h1>
<a href="/2021/01/18/using-rocksdb-state-backend-in-apache-flink-when-and-how/">Using RocksDB State Backend in Apache Flink: When and How</a>
</h1>
January 18, 2021 -
Jun Qin
<p><p>Stream processing applications are often stateful, “remembering” information from processed events and using it to influence further event processing. In Flink, the remembered information, i.e., state, is stored locally in the configured state backend. To prevent data loss in case of failures, the state backend periodically persists a snapshot of its contents to a pre-configured durable storage. The <a href="https://rocksdb.org/">RocksDB</a> state backend (i.e., RocksDBStateBackend) is one of the three built-in state backends in Flink. This blog post will guide you through the benefits of using RocksDB to manage your application’s state, explain when and how to use it and also clear up a few common misconceptions. Having said that, this is <strong>not</strong> a blog post to explain how RocksDB works in-depth or how to do advanced troubleshooting and performance tuning; if you need help with any of those topics, you can reach out to the <a href="https://flink.apache.org/community.html#mailing-lists">Flink User Mailing List</a>.</p>
<h1 id="state-in-flink">
State in Flink
<a class="anchor" href="#state-in-flink">#</a>
</h1>
<p>To best understand state and state backends in Flink, it’s important to distinguish between <strong>in-flight state</strong> and <strong>state snapshots</strong>. In-flight state, also known as working state, is the state a Flink job is working on. It is always stored locally in memory (with the possibility to spill to disk) and can be lost when jobs fail without impacting job recoverability. State snapshots, i.e., <a href="//nightlies.apache.org/flinkflink-docs-stable/ops/state/checkpoints.html">checkpoints</a> and <a href="//nightlies.apache.org/flinkflink-docs-stable/ops/state/savepoints.html#what-is-a-savepoint-how-is-a-savepoint-different-from-a-checkpoint">savepoints</a>, are stored in a remote durable storage, and are used to restore the local state in the case of job failures. The appropriate state backend for a production deployment depends on scalability, throughput, and latency requirements.</p>
<h1 id="what-is-rocksdb">
What is RocksDB?
<a class="anchor" href="#what-is-rocksdb">#</a>
</h1>
<p>Thinking of RocksDB as a distributed database that needs to run on a cluster and to be managed by specialized administrators is a common misconception. RocksDB is an embeddable persistent key-value store for fast storage. It interacts with Flink via the Java Native Interface (JNI). The picture below shows where RocksDB fits in a Flink cluster node. Details are explained in the following sections.</p>
<center>
<img vspace="8" style="width:60%" src="/img/blog/2021-01-18-rocksdb/RocksDB-in-Flink.png" />
</center>
<h1 id="rocksdb-in-flink">
RocksDB in Flink
<a class="anchor" href="#rocksdb-in-flink">#</a>
</h1>
<p>Everything you need to use RocksDB as a state backend is bundled in the Apache Flink distribution, including the native shared library:</p>
<pre><code>$ jar -tvf lib/flink-dist_2.12-1.12.0.jar| grep librocksdbjni-linux64
8695334 Wed Nov 27 02:27:06 CET 2019 librocksdbjni-linux64.so
</code></pre>
<p>At runtime, RocksDB is embedded in the TaskManager processes. It runs in native threads and works with local files. For example, if you have a job configured with RocksDBStateBackend running in your Flink cluster, you’ll see something similar to the following, where 32513 is the TaskManager process ID.</p>
<pre><code>$ ps -T -p 32513 | grep -i rocksdb
32513 32633 ? 00:00:00 rocksdb:low0
32513 32634 ? 00:00:00 rocksdb:high0
</code></pre>
<div class="alert alert-info" markdown="1">
<span class="label label-info" style="display: inline-block"><span class="glyphicon glyphicon-info-sign" aria-hidden="true"></span> Note</span>
The command is for Linux only. For other operating systems, please refer to their documentation.
</div>
<h1 id="when-to-use-rocksdbstatebackend">
When to use RocksDBStateBackend
<a class="anchor" href="#when-to-use-rocksdbstatebackend">#</a>
</h1>
<p>In addition to RocksDBStateBackend, Flink has two other built-in state backends: MemoryStateBackend and FsStateBackend. They both are heap-based, as in-flight state is stored in the JVM heap. For the moment being, let’s ignore MemoryStateBackend, as it is intended only for <strong>local developments</strong> and <strong>debugging</strong>, not for production use.</p>
<p>With RocksDBStateBackend, in-flight state is first written into off-heap/native memory, and then flushed to local disks when a configured threshold is reached. This means that RocksDBStateBackend can support state larger than the total configured heap capacity. The amount of state that you can store in RocksDBStateBackend is only limited by the amount of <strong>disk space</strong> available across the entire cluster. In addition, since RocksDBStateBackend doesn’t use the JVM heap to store in-flight state, it’s not affected by JVM Garbage Collection and therefore has predictable latency.</p>
<p>On top of full, self-contained state snapshots, RocksDBStateBackend also supports <a href="//nightlies.apache.org/flinkflink-docs-stable/ops/state/large_state_tuning.html#incremental-checkpoints">incremental checkpointing</a> as a performance tuning option. An incremental checkpoint stores only the changes that occurred since the latest completed checkpoint. This dramatically reduces checkpointing time in comparison to performing a full snapshot. RocksDBStateBackend is currently the only state backend that supports incremental checkpointing.</p>
<p>RocksDB is a good option when:</p>
<ul>
<li>The state of your job is larger than can fit in local memory (e.g., long windows, large <a href="//nightlies.apache.org/flinkflink-docs-stable/dev/stream/state/state.html">keyed state</a>);</li>
<li>You’re looking into incremental checkpointing as a way to reduce checkpointing time;</li>
<li>You expect to have more predictable latency without being impacted by JVM Garbage Collection.</li>
</ul>
<p>Otherwise, if your application has <strong>small state</strong> or requires <strong>very low latency</strong>, you should consider <strong>FsStateBackend</strong>. As a rule of thumb, RocksDBStateBackend is a few times slower than heap-based state backends, because it stores key/value pairs as serialized bytes. This means that any state access (reads/writes) needs to go through a de-/serialization process crossing the JNI boundary, which is more expensive than working directly with the on-heap representation of state. The upside is that, for the same amount of state, it has a <strong>low memory footprint</strong> compared to the corresponding on-heap representation.</p>
<h1 id="how-to-use-rocksdbstatebackend">
How to use RocksDBStateBackend
<a class="anchor" href="#how-to-use-rocksdbstatebackend">#</a>
</h1>
<p>RocksDB is fully embedded within and fully managed by the TaskManager process. RocksDBStateBackend can be configured at the cluster level as the default for the entire cluster, or at the job level for individual jobs. The job level configuration takes precedence over the cluster level configuration.</p>
<h2 id="cluster-level">
Cluster Level
<a class="anchor" href="#cluster-level">#</a>
</h2>
<p>Add the following configuration in <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html"><code>conf/flink-conf.yaml</code></a>:</p>
<pre><code>state.backend: rocksdb
state.backend.incremental: true
state.checkpoints.dir: hdfs:///flink-checkpoints # location to store checkpoints
</code></pre>
<h1 id="job-level">
Job Level
<a class="anchor" href="#job-level">#</a>
</h1>
<p>Add the following into your job’s code after StreamExecutionEnvironment is created:</p>
<pre><code># 'env' is the created StreamExecutionEnvironment
# 'true' is to enable incremental checkpointing
env.setStateBackend(new RocksDBStateBackend(&quot;hdfs:///fink-checkpoints&quot;, true));
</code></pre>
<div class="alert alert-info" markdown="1">
<span class="label label-info" style="display: inline-block"><span class="glyphicon glyphicon-info-sign" aria-hidden="true"></span> Note</span>
In addition to HDFS, you can also use other on-premises or cloud-based object stores if the corresponding dependencies are added under [FLINK_HOME/plugins](//nightlies.apache.org/flinkflink-docs-stable/deployment/filesystems/plugins.html).
</div>
<h1 id="best-practices-and-advanced-configuration">
Best Practices and Advanced Configuration
<a class="anchor" href="#best-practices-and-advanced-configuration">#</a>
</h1>
<p>We hope this overview helped you gain a better understanding of the role of RocksDB in Flink and how to successfully run a job with RocksDBStateBackend. To round it off, we’ll explore some best practices and a few reference points for further troubleshooting and performance tuning.</p>
<h2 id="state-location-in-rocksdb">
State Location in RocksDB
<a class="anchor" href="#state-location-in-rocksdb">#</a>
</h2>
<p>As mentioned earlier, in-flight state in RocksDBStateBackend is spilled to files on disk. These files are located under the directory specified by the Flink configuration <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-localdir"><code>state.backend.rocksdb.localdir</code></a>. Because disk performance has a direct impact on RocksDB’s performance, it’s recommended that this directory is located on a <strong>local</strong> disk. It’s discouraged to configure it to a remote network-based location like NFS or HDFS, as writing to remote disks is usually slower. Also high availability is not a requirement for in-flight state. Local SSD disks are preferred if high disk throughput is required.</p>
<p>State snapshots are persisted to remote durable storage. During state snapshotting, TaskManagers take a snapshot of the in-flight state and store it remotely. Transferring the state snapshot to remote storage is handled purely by the TaskManager itself without the involvement of the state backend. So, <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-checkpoints-dir"><code>state.checkpoints.dir</code></a> or the parameter you set in the code for a particular job can be different locations like an on-premises <a href="https://hadoop.apache.org/docs/current/hadoop-project-dist/hadoop-hdfs/HdfsDesign.html">HDFS</a> cluster or a cloud-based object store like <a href="https://aws.amazon.com/s3/">Amazon S3</a>, <a href="https://azure.microsoft.com/en-us/services/storage/blobs/">Azure Blob Storage</a>, <a href="https://cloud.google.com/storage">Google Cloud Storage</a>, <a href="https://www.alibabacloud.com/product/oss">Alibaba OSS</a>, etc.</p>
<h2 id="troubleshooting-rocksdb">
Troubleshooting RocksDB
<a class="anchor" href="#troubleshooting-rocksdb">#</a>
</h2>
<p>To check how RocksDB is behaving in production, you should look for the RocksDB log file named LOG. By default, this log file is located in the same directory as your data files, i.e., the directory specified by the Flink configuration <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-localdir"><code>state.backend.rocksdb.localdir</code></a>. When enabled, <a href="https://github.com/facebook/rocksdb/wiki/RocksDB-Tuning-Guide#rocksdb-statistics">RocksDB statistics</a> are also logged there to help diagnose potential problems. For further information, check <a href="https://github.com/facebook/rocksdb/wiki/RocksDB-Troubleshooting-Guide">RocksDB Troubleshooting Guide</a> in <a href="https://github.com/facebook/rocksdb/wiki">RocksDB Wiki</a>. If you are interested in the RocksDB behavior trend over time, you can consider enabling <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#rocksdb-native-metrics">RocksDB native metrics</a> for your Flink job.</p>
<div class="alert alert-info" markdown="1">
<span class="label label-info" style="display: inline-block"><span class="glyphicon glyphicon-info-sign" aria-hidden="true"></span> Note</span>
From Flink 1.10, RocksDB logging was effectively disabled by [setting the log level to HEADER](https://github.com/apache/flink/blob/master/flink-state-backends/flink-statebackend-rocksdb/src/main/java/org/apache/flink/contrib/streaming/state/PredefinedOptions.java#L64). To enable it, check [How to get RocksDB's LOG file back for advanced troubleshooting](https://ververica.zendesk.com/hc/en-us/articles/360015933320-How-to-get-RocksDB-s-LOG-file-back-for-advanced-troubleshooting).
</div>
<div class="alert alert-warning" markdown="1">
<span class="label label-warning" style="display: inline-block"><span class="glyphicon glyphicon-warning-sign" aria-hidden="true"></span> Warning</span>
Enabling RocksDB's native metrics in Flink may have a negative performance impact on your job.
</div>
<h2 id="tuning-rocksdb">
Tuning RocksDB
<a class="anchor" href="#tuning-rocksdb">#</a>
</h2>
<p>Since Flink 1.10, Flink configures RocksDB’s memory allocation to the amount of managed memory of each task slot by default. The primary mechanism for improving memory-related performance issues is to increase Flink’s <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/memory/mem_setup_tm.html#managed-memory">managed memory</a> via the Flink configuration <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#taskmanager-memory-managed-size"><code>taskmanager.memory.managed.size</code></a> or <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#taskmanager-memory-managed-fraction"><code>taskmanager.memory.managed.fraction</code></a>. For more fine-grained control, you should first disable the automatic memory management by setting <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-memory-managed"><code>state.backend.rocksdb.memory.managed</code></a> to <code>false</code>, then start with the following Flink configuration: <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-block-cache-size"><code>state.backend.rocksdb.block.cache-size</code></a> (corresponding to block_cache_size in RocksDB), <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-writebuffer-size"><code>state.backend.rocksdb.writebuffer.size</code></a> (corresponding to write_buffer_size in RocksDB), and <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-writebuffer-count"><code>state.backend.rocksdb.writebuffer.count</code></a> (corresponding to max_write_buffer_number in RocksDB). For more details, check <a href="https://www.ververica.com/blog/manage-rocksdb-memory-size-apache-flink">this blog post</a> on how to manage RocksDB memory size in Flink and the <a href="https://github.com/facebook/rocksdb/wiki/Memory-usage-in-RocksDB">RocksDB Memory Usage</a> Wiki page.</p>
<p>While data is being written or overwritten in RocksDB, flushing from memory to local disks and data compaction are managed in the background by RocksDB threads. On a machine with many CPU cores, you should increase the parallelism of background flushing and compaction by setting the Flink configuration <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#state-backend-rocksdb-thread-num"><code>state.backend.rocksdb.thread.num</code></a> (corresponding to max_background_jobs in RocksDB). The default configuration is usually too small for a production setup. If your job reads frequently from RocksDB, you should consider enabling <a href="https://github.com/facebook/rocksdb/wiki/RocksDB-Tuning-Guide#bloom-filters">bloom filters</a>.</p>
<p>For other RocksDBStateBackend configurations, check the Flink documentation on <a href="//nightlies.apache.org/flinkflink-docs-stable/deployment/config.html#advanced-rocksdb-state-backends-options">Advanced RocksDB State Backends Options</a>. For further tuning, check <a href="https://github.com/facebook/rocksdb/wiki/RocksDB-Tuning-Guide">RocksDB Tuning Guide</a> in <a href="https://github.com/facebook/rocksdb/wiki">RocksDB Wiki</a>.</p>
<h1 id="conclusion">
Conclusion
<a class="anchor" href="#conclusion">#</a>
</h1>
<p>The <a href="https://rocksdb.org/">RocksDB</a> state backend (i.e., RocksDBStateBackend) is one of the three state backends bundled in Flink, and can be a powerful choice when configuring your streaming applications. It enables scalable applications maintaining up to many terabytes of state with exactly-once processing guarantees. If the state of your Flink job is too large to fit on the JVM heap, you are interested in incremental checkpointing, or you expect to have predictable latency, you should use RocksDBStateBackend. Since RocksDB is embedded in TaskManager processes as native threads and works with files on local disks, RocksDBStateBackend is supported out-of-the-box without the need to further setup and manage any external systems or processes.</p>
</p>
</article>
<footer class="book-footer">
<a href="https://cwiki.apache.org/confluence/display/FLINK/Flink+Translation+Specifications">Want to contribute translation?</a>
<br><br>
<a href="//github.com/apache/flink-web/edit/asf-site/docs/content/posts/2021-01-18-rocksdb.md" style="color:black"><i class="fa fa-edit fa-fw"></i>Edit This Page</a>
</footer>
<div class="book-comments">
</div>
<label for="menu-control" class="hidden book-menu-overlay"></label>
</div>
<aside class="book-toc">
<nav id="TableOfContents"><h3>On This Page <button class="toc" onclick="collapseToc()"><i class="fa fa-compress" aria-hidden="true"></i></button></h3>
<ul>
<li><a href="#cluster-level">Cluster Level</a></li>
</ul>
<ul>
<li><a href="#state-location-in-rocksdb">State Location in RocksDB</a></li>
<li><a href="#troubleshooting-rocksdb">Troubleshooting RocksDB</a></li>
<li><a href="#tuning-rocksdb">Tuning RocksDB</a></li>
</ul>
</nav>
</aside>
<aside class="expand-toc">
<button class="toc" onclick="expandToc()">
<i class="fa fa-expand" aria-hidden="true"></i>
</button>
</aside>
</main>
</body>
</html>