| |
| <!DOCTYPE html> |
| <html lang="en" dir=> |
| |
| <head> |
| <meta name="generator" content="Hugo 0.111.3"> |
| <meta charset="UTF-8"> |
| <meta name="viewport" content="width=device-width, initial-scale=1.0"> |
| <meta name="description" content="In the first article of the series, we gave a high-level description of the objectives and required functionality of a Fraud Detection engine. We also described how to make data partitioning in Apache Flink customizable based on modifiable rules instead of using a hardcoded KeysExtractor implementation. |
| We intentionally omitted details of how the applied rules are initialized and what possibilities exist for updating them at runtime. In this post, we will address exactly these details."> |
| <meta name="theme-color" content="#FFFFFF"><meta property="og:title" content="Advanced Flink Application Patterns Vol.2: Dynamic Updates of Application Logic" /> |
| <meta property="og:description" content="In the first article of the series, we gave a high-level description of the objectives and required functionality of a Fraud Detection engine. We also described how to make data partitioning in Apache Flink customizable based on modifiable rules instead of using a hardcoded KeysExtractor implementation. |
| We intentionally omitted details of how the applied rules are initialized and what possibilities exist for updating them at runtime. In this post, we will address exactly these details." /> |
| <meta property="og:type" content="article" /> |
| <meta property="og:url" content="https://flink.apache.org/2020/03/24/advanced-flink-application-patterns-vol.2-dynamic-updates-of-application-logic/" /><meta property="article:section" content="posts" /> |
| <meta property="article:published_time" content="2020-03-24T12:00:00+00:00" /> |
| <meta property="article:modified_time" content="2020-03-24T12:00:00+00:00" /> |
| <title>Advanced Flink Application Patterns Vol.2: Dynamic Updates of Application Logic | 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+28Iyvkdhgf5o="> |
| <script defer src="/en.search.min.8592fd2e43835d2ef6fab8eb9b8969ee6ad1bdb888a636e37e28032f8bd9887d.js" integrity="sha256-hZL9LkODXS72+rjrm4lp7mrRvbiIpjbjfigDL4vZiH0="></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/2020/03/24/advanced-flink-application-patterns-vol.2-dynamic-updates-of-application-logic/"> |
| |
| |
| <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> |
| 中文版 |
| </a> |
| |
| <script src="/js/track-search-terms.js"></script> |
| |
| |
| </nav> |
| |
| |
| |
| |
| <script>(function(){var e=document.querySelector("aside.book-menu nav");addEventListener("beforeunload",function(){localStorage.setItem("menu.scrollTop",e.scrollTop)}),e.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>Advanced Flink Application Patterns Vol.2: Dynamic Updates of Application Logic</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="#rules-broadcasting">Rules Broadcasting</a></li> |
| <li><a href="#data-exchange-inside-apache-flink">Data Exchange inside Apache Flink</a></li> |
| <li><a href="#broadcast-state-pattern">Broadcast State Pattern</a></li> |
| </ul> |
| </nav> |
| |
| |
| </aside> |
| |
| |
| </header> |
| |
| |
| |
| |
| |
| |
| |
| <article class="markdown"> |
| <h1> |
| <a href="/2020/03/24/advanced-flink-application-patterns-vol.2-dynamic-updates-of-application-logic/">Advanced Flink Application Patterns Vol.2: Dynamic Updates of Application Logic</a> |
| </h1> |
| |
| March 24, 2020 - |
| |
| |
| |
| Alexander Fedulov |
| |
| <a href="https://twitter.com/alex_fedulov">(@alex_fedulov)</a> |
| |
| |
| |
| |
| <p><p>In the <a href="https://flink.apache.org/news/2020/01/15/demo-fraud-detection.html">first article</a> of the series, we gave a high-level description of the objectives and required functionality of a Fraud Detection engine. We also described how to make data partitioning in Apache Flink customizable based on modifiable rules instead of using a hardcoded <code>KeysExtractor</code> implementation.</p> |
| <p>We intentionally omitted details of how the applied rules are initialized and what possibilities exist for updating them at runtime. In this post, we will address exactly these details. You will learn how the approach to data partitioning described in <a href="https://flink.apache.org/news/2020/01/15/demo-fraud-detection.html">Part 1</a> can be applied in combination with a dynamic configuration. These two patterns, when used together, can eliminate the need to recompile the code and redeploy your Flink job for a wide range of modifications of the business logic.</p> |
| <h2 id="rules-broadcasting"> |
| Rules Broadcasting |
| <a class="anchor" href="#rules-broadcasting">#</a> |
| </h2> |
| <p>Let’s first have a look at the <a href="https://flink.apache.org/news/2020/01/15/demo-fraud-detection.html#dynamic-data-partitioning">previously-defined</a> data-processing pipeline:</p> |
| <div class="highlight"><pre tabindex="0" class="chroma"><code class="language-java" data-lang="java"><span class="line"><span class="cl"><span class="n">DataStream</span><span class="o"><</span><span class="n">Alert</span><span class="o">></span> <span class="n">alerts</span> <span class="o">=</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">transactions</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">process</span><span class="o">(</span><span class="k">new</span> <span class="n">DynamicKeyFunction</span><span class="o">())</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">keyBy</span><span class="o">((</span><span class="n">keyed</span><span class="o">)</span> <span class="o">-></span> <span class="n">keyed</span><span class="o">.</span><span class="na">getKey</span><span class="o">());</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">process</span><span class="o">(</span><span class="k">new</span> <span class="n">DynamicAlertFunction</span><span class="o">())</span> |
| </span></span></code></pre></div><p><code>DynamicKeyFunction</code> provides dynamic data partitioning while <code>DynamicAlertFunction</code> is responsible for executing the main logic of processing transactions and sending alert messages according to defined rules.</p> |
| <p>Vol.1 of this series simplified the use case and assumed that the applied set of rules is pre-initialized and accessible via the <code>List<Rules></code> within <code>DynamicKeyFunction</code>.</p> |
| <div class="highlight"><pre tabindex="0" class="chroma"><code class="language-java" data-lang="java"><span class="line"><span class="cl"><span class="kd">public</span> <span class="kd">class</span> <span class="nc">DynamicKeyFunction</span> |
| </span></span><span class="line"><span class="cl"> <span class="kd">extends</span> <span class="n">ProcessFunction</span><span class="o"><</span><span class="n">Transaction</span><span class="o">,</span> <span class="n">Keyed</span><span class="o"><</span><span class="n">Transaction</span><span class="o">,</span> <span class="n">String</span><span class="o">,</span> <span class="n">Integer</span><span class="o">>></span> <span class="o">{</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"> <span class="cm">/* Simplified */</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">List</span><span class="o"><</span><span class="n">Rule</span><span class="o">></span> <span class="n">rules</span> <span class="o">=</span> <span class="cm">/* Rules that are initialized somehow.*/</span><span class="o">;</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">...</span> |
| </span></span><span class="line"><span class="cl"><span class="o">}</span> |
| </span></span></code></pre></div><p>Adding rules to this list is obviously possible directly inside the code of the Flink Job at the stage of its initialization (Create a <code>List</code> object; use it’s <code>add</code> method). A major drawback of doing so is that it will require recompilation of the job with each rule modification. In a real Fraud Detection system, rules are expected to change on a frequent basis, making this approach unacceptable from the point of view of business and operational requirements. A different approach is needed.</p> |
| <p>Next, let’s take a look at a sample rule definition that we introduced in the previous post of the series:</p> |
| <center> |
| <img src="/img/blog/patterns-blog-2/rule-dsl.png" width="800px" alt="Figure 1: Rule definition"/> |
| <br/> |
| <i><small>Figure 1: Rule definition</small></i> |
| </center> |
| <br/> |
| <p>The previous post covered use of <code>groupingKeyNames</code> by <code>DynamicKeyFunction</code> to extract message keys. Parameters from the second part of this rule are used by <code>DynamicAlertFunction</code>: they define the actual logic of the performed operations and their parameters (such as the alert-triggering limit). This means that the same rule must be present in both <code>DynamicKeyFunction</code> and <code>DynamicAlertFunction</code>. To achieve this result, we will use the <a href="//nightlies.apache.org/flinkflink-docs-release-1.10/dev/stream/state/broadcast_state.html">broadcast data distribution mechanism</a> of Apache Flink.</p> |
| <p>Figure 2 presents the final job graph of the system that we are building:</p> |
| <center> |
| <img src="/img/blog/patterns-blog-2/job-graph.png" width="800px" alt="Figure 2: Job Graph of the Fraud Detection Flink Job"/> |
| <br/> |
| <i><small>Figure 2: Job Graph of the Fraud Detection Flink Job</small></i> |
| </center> |
| <br/> |
| <p>The main blocks of the Transactions processing pipeline are:<br></p> |
| <ul> |
| <li> |
| <p><strong>Transaction Source</strong> that consumes transaction messages from Kafka partitions in parallel. <br></p> |
| </li> |
| <li> |
| <p><strong>Dynamic Key Function</strong> that performs data enrichment with a dynamic key. The subsequent <code>keyBy</code> hashes this dynamic key and partitions the data accordingly among all parallel instances of the following operator.</p> |
| </li> |
| <li> |
| <p><strong>Dynamic Alert Function</strong> that accumulates a data window and creates Alerts based on it.</p> |
| </li> |
| </ul> |
| <h2 id="data-exchange-inside-apache-flink"> |
| Data Exchange inside Apache Flink |
| <a class="anchor" href="#data-exchange-inside-apache-flink">#</a> |
| </h2> |
| <p>The job graph above also indicates various data exchange patterns between the operators. In order to understand how the broadcast pattern works, let’s take a short detour and discuss what methods of message propagation exist in Apache Flink’s distributed runtime.</p> |
| <ul> |
| <li>The <strong>FORWARD</strong> connection after the Transaction Source means that all data consumed by one of the parallel instances of the Transaction Source operator is transferred to exactly one instance of the subsequent <code>DynamicKeyFunction</code> operator. It also indicates the same level of parallelism of the two connected operators (12 in the above case). This communication pattern is illustrated in Figure 3. Orange circles represent transactions, and dotted rectangles depict parallel instances of the conjoined operators.</li> |
| </ul> |
| <center> |
| <img src="/img/blog/patterns-blog-2/forward.png" width="800px" alt="Figure 3: FORWARD message passing across operator instances"/> |
| <br/> |
| <i><small>Figure 3: FORWARD message passing across operator instances</small></i> |
| </center> |
| <br/> |
| <ul> |
| <li>The <strong>HASH</strong> connection between <code>DynamicKeyFunction</code> and <code>DynamicAlertFunction</code> means that for each message a hash code is calculated and messages are evenly distributed among available parallel instances of the next operator. Such a connection needs to be explicitly “requested” from Flink by using <code>keyBy</code>.</li> |
| </ul> |
| <center> |
| <img src="/img/blog/patterns-blog-2/hash.png" width="800px" alt="Figure 4: HASHED message passing across operator instances (via `keyBy`)"/> |
| <br/> |
| <i><small>Figure 4: HASHED message passing across operator instances (via `keyBy`)</small></i> |
| </center> |
| <br/> |
| <ul> |
| <li>A <strong>REBALANCE</strong> distribution is either caused by an explicit call to <code>rebalance()</code> or by a change of parallelism (12 -> 1 in the case of the job graph from Figure 2). Calling <code>rebalance()</code> causes data to be repartitioned in a round-robin fashion and can help to mitigate data skew in certain scenarios.</li> |
| </ul> |
| <center> |
| <img src="/img/blog/patterns-blog-2/rebalance.png" width="800px" alt="Figure 5: REBALANCE message passing across operator instances"/> |
| <br/> |
| <i><small>Figure 5: REBALANCE message passing across operator instances</small></i> |
| </center> |
| <br/> |
| <p>The Fraud Detection job graph in Figure 2 contains an additional data source: <em>Rules Source</em>. It also consumes from Kafka. Rules are “mixed into” the main processing data flow through the <strong>BROADCAST</strong> channel. Unlike other methods of transmitting data between operators, such as <code>forward</code>, <code>hash</code> or <code>rebalance</code> that make each message available for processing in only one of the parallel instances of the receiving operator, <code>broadcast</code> makes each message available at the input of all of the parallel instances of the operator to which the <em>broadcast stream</em> is connected. This makes <code>broadcast</code> applicable to a wide range of tasks that need to affect the processing of all messages, regardless of their key or source partition.</p> |
| <center> |
| <img src="/img/blog/patterns-blog-2/broadcast.png" width="800px" alt="Figure 6: BROADCAST message passing across operator instances"/> |
| <br/> |
| <i><small>Figure 6: BROADCAST message passing across operator instances</small></i> |
| </center> |
| <br/> |
| <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> |
| There are actually a few more specialized data partitioning schemes in Flink which we did not mention here. If you want to find out more, please refer to Flink's documentation on __[stream partitioning](//nightlies.apache.org/flinkflink-docs-stable/dev/stream/operators/#physical-partitioning)__. |
| </div> |
| <h2 id="broadcast-state-pattern"> |
| Broadcast State Pattern |
| <a class="anchor" href="#broadcast-state-pattern">#</a> |
| </h2> |
| <p>In order to make use of the Rules Source, we need to “connect” it to the main data stream:</p> |
| <div class="highlight"><pre tabindex="0" class="chroma"><code class="language-java" data-lang="java"><span class="line"><span class="cl"><span class="c1">// Streams setup |
| </span></span></span><span class="line"><span class="cl"><span class="c1"></span><span class="n">DataStream</span><span class="o"><</span><span class="n">Transaction</span><span class="o">></span> <span class="n">transactions</span> <span class="o">=</span> <span class="o">[...]</span> |
| </span></span><span class="line"><span class="cl"><span class="n">DataStream</span><span class="o"><</span><span class="n">Rule</span><span class="o">></span> <span class="n">rulesUpdateStream</span> <span class="o">=</span> <span class="o">[...]</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"><span class="n">BroadcastStream</span><span class="o"><</span><span class="n">Rule</span><span class="o">></span> <span class="n">rulesStream</span> <span class="o">=</span> <span class="n">rulesUpdateStream</span><span class="o">.</span><span class="na">broadcast</span><span class="o">(</span><span class="n">RULES_STATE_DESCRIPTOR</span><span class="o">);</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"><span class="c1">// Processing pipeline setup |
| </span></span></span><span class="line"><span class="cl"><span class="c1"></span> <span class="n">DataStream</span><span class="o"><</span><span class="n">Alert</span><span class="o">></span> <span class="n">alerts</span> <span class="o">=</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">transactions</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">connect</span><span class="o">(</span><span class="n">rulesStream</span><span class="o">)</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">process</span><span class="o">(</span><span class="k">new</span> <span class="n">DynamicKeyFunction</span><span class="o">())</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">keyBy</span><span class="o">((</span><span class="n">keyed</span><span class="o">)</span> <span class="o">-></span> <span class="n">keyed</span><span class="o">.</span><span class="na">getKey</span><span class="o">())</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">connect</span><span class="o">(</span><span class="n">rulesStream</span><span class="o">)</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">.</span><span class="na">process</span><span class="o">(</span><span class="k">new</span> <span class="n">DynamicAlertFunction</span><span class="o">())</span> |
| </span></span></code></pre></div><p>As you can see, the broadcast stream can be created from any regular stream by calling the <code>broadcast</code> method and specifying a state descriptor. Flink assumes that broadcasted data needs to be stored and retrieved while processing events of the main data flow and, therefore, always automatically creates a corresponding <em>broadcast state</em> from this state descriptor. This is different from any other Apache Flink state type in which you need to initialize it in the <code>open()</code> method of the processing function. Also note that broadcast state always has a key-value format (<code>MapState</code>).</p> |
| <div class="highlight"><pre tabindex="0" class="chroma"><code class="language-java" data-lang="java"><span class="line"><span class="cl"><span class="kd">public</span> <span class="kd">static</span> <span class="kd">final</span> <span class="n">MapStateDescriptor</span><span class="o"><</span><span class="n">Integer</span><span class="o">,</span> <span class="n">Rule</span><span class="o">></span> <span class="n">RULES_STATE_DESCRIPTOR</span> <span class="o">=</span> |
| </span></span><span class="line"><span class="cl"> <span class="k">new</span> <span class="n">MapStateDescriptor</span><span class="o"><>(</span><span class="s">"rules"</span><span class="o">,</span> <span class="n">Integer</span><span class="o">.</span><span class="na">class</span><span class="o">,</span> <span class="n">Rule</span><span class="o">.</span><span class="na">class</span><span class="o">);</span> |
| </span></span></code></pre></div><p>Connecting to <code>rulesStream</code> causes some changes in the signature of the processing functions. The previous article presented it in a slightly simplified way as a <code>ProcessFunction</code>. However, <code>DynamicKeyFunction</code> is actually a <code>BroadcastProcessFunction</code>.</p> |
| <div class="highlight"><pre tabindex="0" class="chroma"><code class="language-java" data-lang="java"><span class="line"><span class="cl"><span class="kd">public</span> <span class="kd">abstract</span> <span class="kd">class</span> <span class="nc">BroadcastProcessFunction</span><span class="o"><</span><span class="n">IN1</span><span class="o">,</span> <span class="n">IN2</span><span class="o">,</span> <span class="n">OUT</span><span class="o">></span> <span class="o">{</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"> <span class="kd">public</span> <span class="kd">abstract</span> <span class="kt">void</span> <span class="nf">processElement</span><span class="o">(</span><span class="n">IN1</span> <span class="n">value</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">ReadOnlyContext</span> <span class="n">ctx</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">Collector</span><span class="o"><</span><span class="n">OUT</span><span class="o">></span> <span class="n">out</span><span class="o">)</span> <span class="kd">throws</span> <span class="n">Exception</span><span class="o">;</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"> <span class="kd">public</span> <span class="kd">abstract</span> <span class="kt">void</span> <span class="nf">processBroadcastElement</span><span class="o">(</span><span class="n">IN2</span> <span class="n">value</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">Context</span> <span class="n">ctx</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">Collector</span><span class="o"><</span><span class="n">OUT</span><span class="o">></span> <span class="n">out</span><span class="o">)</span> <span class="kd">throws</span> <span class="n">Exception</span><span class="o">;</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"><span class="o">}</span> |
| </span></span></code></pre></div><p>The difference is the addition of the <code>processBroadcastElement</code> method through which messages of the rules stream will arrive. The following new version of <code>DynamicKeyFunction</code> allows modifying the list of data-distribution keys at runtime through this stream:</p> |
| <div class="highlight"><pre tabindex="0" class="chroma"><code class="language-java" data-lang="java"><span class="line"><span class="cl"><span class="kd">public</span> <span class="kd">class</span> <span class="nc">DynamicKeyFunction</span> |
| </span></span><span class="line"><span class="cl"> <span class="kd">extends</span> <span class="n">BroadcastProcessFunction</span><span class="o"><</span><span class="n">Transaction</span><span class="o">,</span> <span class="n">Rule</span><span class="o">,</span> <span class="n">Keyed</span><span class="o"><</span><span class="n">Transaction</span><span class="o">,</span> <span class="n">String</span><span class="o">,</span> <span class="n">Integer</span><span class="o">>></span> <span class="o">{</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"> <span class="nd">@Override</span> |
| </span></span><span class="line"><span class="cl"> <span class="kd">public</span> <span class="kt">void</span> <span class="nf">processBroadcastElement</span><span class="o">(</span><span class="n">Rule</span> <span class="n">rule</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">Context</span> <span class="n">ctx</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">Collector</span><span class="o"><</span><span class="n">Keyed</span><span class="o"><</span><span class="n">Transaction</span><span class="o">,</span> <span class="n">String</span><span class="o">,</span> <span class="n">Integer</span><span class="o">>></span> <span class="n">out</span><span class="o">)</span> <span class="o">{</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">BroadcastState</span><span class="o"><</span><span class="n">Integer</span><span class="o">,</span> <span class="n">Rule</span><span class="o">></span> <span class="n">broadcastState</span> <span class="o">=</span> <span class="n">ctx</span><span class="o">.</span><span class="na">getBroadcastState</span><span class="o">(</span><span class="n">RULES_STATE_DESCRIPTOR</span><span class="o">);</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">broadcastState</span><span class="o">.</span><span class="na">put</span><span class="o">(</span><span class="n">rule</span><span class="o">.</span><span class="na">getRuleId</span><span class="o">(),</span> <span class="n">rule</span><span class="o">);</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">}</span> |
| </span></span><span class="line"><span class="cl"> |
| </span></span><span class="line"><span class="cl"> <span class="nd">@Override</span> |
| </span></span><span class="line"><span class="cl"> <span class="kd">public</span> <span class="kt">void</span> <span class="nf">processElement</span><span class="o">(</span><span class="n">Transaction</span> <span class="n">event</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">ReadOnlyContext</span> <span class="n">ctx</span><span class="o">,</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">Collector</span><span class="o"><</span><span class="n">Keyed</span><span class="o"><</span><span class="n">Transaction</span><span class="o">,</span> <span class="n">String</span><span class="o">,</span> <span class="n">Integer</span><span class="o">>></span> <span class="n">out</span><span class="o">){</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">ReadOnlyBroadcastState</span><span class="o"><</span><span class="n">Integer</span><span class="o">,</span> <span class="n">Rule</span><span class="o">></span> <span class="n">rulesState</span> <span class="o">=</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">ctx</span><span class="o">.</span><span class="na">getBroadcastState</span><span class="o">(</span><span class="n">RULES_STATE_DESCRIPTOR</span><span class="o">);</span> |
| </span></span><span class="line"><span class="cl"> <span class="k">for</span> <span class="o">(</span><span class="n">Map</span><span class="o">.</span><span class="na">Entry</span><span class="o"><</span><span class="n">Integer</span><span class="o">,</span> <span class="n">Rule</span><span class="o">></span> <span class="n">entry</span> <span class="o">:</span> <span class="n">rulesState</span><span class="o">.</span><span class="na">immutableEntries</span><span class="o">())</span> <span class="o">{</span> |
| </span></span><span class="line"><span class="cl"> <span class="kd">final</span> <span class="n">Rule</span> <span class="n">rule</span> <span class="o">=</span> <span class="n">entry</span><span class="o">.</span><span class="na">getValue</span><span class="o">();</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">out</span><span class="o">.</span><span class="na">collect</span><span class="o">(</span> |
| </span></span><span class="line"><span class="cl"> <span class="k">new</span> <span class="n">Keyed</span><span class="o"><>(</span> |
| </span></span><span class="line"><span class="cl"> <span class="n">event</span><span class="o">,</span> <span class="n">KeysExtractor</span><span class="o">.</span><span class="na">getKey</span><span class="o">(</span><span class="n">rule</span><span class="o">.</span><span class="na">getGroupingKeyNames</span><span class="o">(),</span> <span class="n">event</span><span class="o">),</span> <span class="n">rule</span><span class="o">.</span><span class="na">getRuleId</span><span class="o">()));</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">}</span> |
| </span></span><span class="line"><span class="cl"> <span class="o">}</span> |
| </span></span><span class="line"><span class="cl"><span class="o">}</span> |
| </span></span></code></pre></div><p>In the above code, <code>processElement()</code> receives Transactions, and <code>processBroadcastElement()</code> receives Rule updates. When a new rule is created, it is distributed as depicted in Figure 6 and saved in all parallel instances of the operator using <code>processBroadcastState</code>. We use a Rule’s ID as the key to store and reference individual rules. Instead of iterating over a hardcoded <code>List<Rules></code>, we iterate over entries in the dynamically-updated broadcast state.</p> |
| <p><code>DynamicAlertFunction</code> follows the same logic with respect to storing the rules in the broadcast <code>MapState</code>. As described in <a href="https://flink.apache.org/news/2020/01/15/demo-fraud-detection.html">Part 1</a>, each message in the <code>processElement</code> input is intended to be processed by one specific rule and comes “pre-marked” with a corresponding ID by <code>DynamicKeyFunction</code>. All we need to do is retrieve the definition of the corresponding rule from <code>BroadcastState</code> by using the provided ID and process it according to the logic required by that rule. At this stage, we will also add messages to the internal function state in order to perform calculations on the required time window of data. We will consider how this is done in the <a href="/news/2020/07/30/demo-fraud-detection-3.html">final blog</a> of the series about Fraud Detection.</p> |
| <h1 id="summary"> |
| Summary |
| <a class="anchor" href="#summary">#</a> |
| </h1> |
| <p>In this blog post, we continued our investigation of the use case of a Fraud Detection System built with Apache Flink. We looked into different ways in which data can be distributed between parallel operator instances and, most importantly, examined broadcast state. We demonstrated how dynamic partitioning — a pattern described in the <a href="https://flink.apache.org/news/2020/01/15/demo-fraud-detection.html">first part</a> of the series — can be combined and enhanced by the functionality provided by the broadcast state pattern. The ability to send dynamic updates at runtime is a powerful feature of Apache Flink that is applicable in a variety of other use cases, such as controlling state (cleanup/insert/fix), running A/B experiments or executing updates of ML model coefficients.</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/2020-03-24-demo-fraud-detection-2.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="#rules-broadcasting">Rules Broadcasting</a></li> |
| <li><a href="#data-exchange-inside-apache-flink">Data Exchange inside Apache Flink</a></li> |
| <li><a href="#broadcast-state-pattern">Broadcast State Pattern</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> |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |