使用 Alluxio 优化 EMR 上 Flink Join

0
0
{"value":"\n\n#### **业务背景&痛点**\n\n\n- 流式处理的业务场景,经常会遇到实时消息数据需要与历史存量数据关联查询或者聚合,比如电商常见的订单场景,订单表做为实时事实表,是典型的流式消息数据,通常会在 kafka 中,而客户信息,商品 SKU 表是维度表,通常存在业务数据库或者数仓中,是典型的离线数据。实时订单数据在实时处理时通常需要事实表与维度表 join 做 reference 补全,以便拿到订单详情并实时统计当天或截至当天的所有订单的商品分布详情。\n- 流式计算通常采用 Flink 做为数据处理平台,上文中提到的实时和离线数据join 的场景,Flink 提供了 Hive/ jdbc/Hudi/ filesystem 各种 connector 实现与离线数据的提取和读写,这样一来在 Flink 应用程序中,即可使用 Table,Sql API 来 join 关联流态表和离线表数据,实现聚合计算等操作\n\n使用 Flink Sql 离线表 Join 流态表的常规 lookup join,是通过 Flink hive sql connector 或者 filesystem connector,对离线 hive 库表或者 S3上离线数据建 Flink Table,然后对 kafka 消息流中的数据建流态表,然后直接做量表做 join 操作\n\n该方式架构如下图所示:\n\n![image.png](https://dev-media.amazoncloud.cn/18381b288b3f4762a86704a4afd2cea1_image.png)\n\n该方式主要面临的问题是:\n\n- lookup 维度表数据只会在首次拉起 Flink 应用的时候,保存在 task manager state 中,后续持续查询或者开窗聚合等操作时,是不会再次拉取维度表数据,业务需要定期重启 Flink 应用,或者刷新维度表数据到临时表,以便 join 聚合时和最新的维度数据关联:\n\n![image.png](https://dev-media.amazoncloud.cn/c75ea2eb525f4560bc5eaa5bd16729ac_image.png)\n\n- 每次需要重新全量拉取维度表数据,存在冷启动问题,且维度表数据量大的时候(如上千万注册用户信息表,上万的商品 SKU 属性字段),造成很大 IO 开销,存在性能瓶颈\n- Flink 的 checkpoint 机制在持续查询或者开窗聚合时,需要保存 state 状态及处理数据到检查点快照中,造成 state 快照数据膨胀\n\n\n#### **解决方案思路**\n\n\n基于以上业务难点,本文提出一种解决方案思路,即通过 Alluxio 缓存层,将 hive 维度表数据自动加载至 Alluxio UFS 缓存中,同时通过 Flink 时态表 join,把维度表数据做成持续变化表上某一时刻的视图\n\n同时使用 Flink 的 Temporal table function 表函数,传递一个时间参数,返回 Temporal table 这一指定时刻的视图,这样实时动态表主表与这个 Temporal table 表关联的时候,可以关联到某一个版本(历史上某一个时刻)的维度数据\n\n优化后的整体架构如下图所示:\n\n![image.png](https://dev-media.amazoncloud.cn/4b84fb4f6b9f4dd08dc1d1117712ecc1_image.png)\n\n\n#### **方案实施落地Detail**\n\n\n本文以 Kafka 中用户行为日志数据做为实时流态的事实表数据,hive 上用户信息数据做为离线维度表数据,采用 Alluxio+Flink temproal 的 demo,来验证其 flink join 优化的解决方案\n\n\n##### **实时事实表**\n\n\n本实例中我们使用 json-data-generator 开源组件模拟的用户行为 json 数据,实时写入 kafka 中,通过 Flink kafka connector 转换为持续查询的 Flink 流态表,从而做为实时 join 的时候的 Fact 事实表数据\n\n用户行为 json 模拟数据如下格式:\n\n```\\n[{ \\"timestamp\\": \\"nowTimestamp()\\",\\n \\"system\\": \\"BADGE\\",\\n \\"actor\\": \\"Agnew\\",\\n \\"action\\": \\"EXIT\\",\\n \\"objects\\": [\\"Building 1\\"],\\n \\"location\\": \\"45.5,44.3\\",\\n \\"message\\": \\"Exited Building 1\\"\\n }]\\n```\n\n包含用户行为的业务时间,登录系统,用户署名,行为 activity 动作,操作涉及对象,位置信息,及相关文本消息字段。我们在\n\nFlink Sql 中建选择主要字段建事实表如下\n\n```\\nCREATE TABLE logevent_source (`timestamp` string, \\n`system` string,\\n actor STRING,\\n action STRING\\n) WITH (\\n'connector' = 'kafka',\\n'topic' = 'logevent',\\n'properties.bootstrap.servers' = 'b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092 (http://b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092/)',\\n'properties.group.id' = 'testGroup6',\\n'scan.startup.mode'='latest-offset',\\n'format' = 'json'\\n);\\n```\n\n\n##### **Alluxio 缓存维度表**\n\n\nAlluxio 是大数据技术堆栈的分布式缓存,它提供了一个统一的 UFS 文件系统可以对接底层 S3,hdfs 数据,在读写 Alluxio UFS 的时候,可以针对 S3,HDFS 分布式存储层实现 warm up,显著提升吞吐量和减少网络开销,且与上层计算引擎如 Hive,spark,Trino 都有深度的集成,很适合做为离线维度数据的缓存加速器\n\n[Amazon EMR](https://aws.amazon.com/cn/emr/?trk=cndc-detail) 对 Alluxio 提供了良好的集成,可以通过 boostrap 启动脚本方式,在 EMR 创建时自动部署 Alluxio 组件并启动 Alluxio master、worker 进程,详细 EMR 安装和部署 Alluxio 步骤可以参考另一篇文章 [Alluxio EMR 集成实践](https://quip-amazon.com/zddcAHEvMDvW)\n\n在集成 Alluxio 的 [Amazon EMR](https://aws.amazon.com/cn/emr/?trk=cndc-detail) 集群中,使用 Alluxio 中创建 hive 离线维表数据的缓存表方法如下:\n\n```\\nhive-env.sh中设置设置client jar包:\\n\$ export HIVE_AUX_JARS_PATH=/<PATH_TO_ALLUXIO>/client/alluxio-2.2.0-client.jar:\${HIVE_AU\\n\\n确保安装部署alluxio的EMR集群上ufs已配置,并且表或者db路径已创建\\nalluxio fs mkdir alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/customer\\nalluxio fs chown hadoop:hadoop alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/customer\\n\\n在AWS EMR集群上,创建hive表路径指向alluxio namespace uri:\\n!connect jdbc:hive2://xxx.xxx.xxx.xxx:10000/default;\\nhive> CREATE TABLE customer(\\n c_customer_sk bigint,\\n c_customer_id string,\\n c_current_cdemo_sk bigint,\\n c_current_hdemo_sk bigint,\\n c_current_addr_sk bigint,\\n c_first_shipto_date_sk bigint,\\n c_first_sales_date_sk bigint,\\n c_salutation string,\\n c_first_name string,\\n c_last_name string,\\n c_preferred_cust_flag string,\\n c_birth_day int,\\n c_birth_month int,\\n c_birth_year int,\\n c_birth_country string,\\n c_login string,\\n c_email_address string\\n)\\n ROW FORMAT DELIMITED\\n FIELDS TERMINATED BY '|'\\n STORED AS TEXTFILE\\n LOCATION 'alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/customer';\\nOK\\nTime taken: 3.485 seconds\\n```\n\n如上所示,该 Alluxio 表 location 指向的路径即为 hive 维度表所在 S3路径,因此对 Customer 用户维度信息表的写入操作会自动同步到 alluxio 缓存中。\n\n创建好 Alluxio hive 离线维度表后,在 flink sql中,可以通过 hive 的 catalog,连接到 hive 元数据,即可以查看到 alluxio 缓存表的详细信息:\n\n```\\nCREATE CATALOG hiveCatalog WITH ( 'type' = 'hive',\\n 'default-database' = 'default',\\n 'hive-conf-dir' = '/etc/hive/conf/',\\n 'hive-version' = '3.1.2',\\n 'hadoop-conf-dir'='/etc/hadoop/conf/'\\n);\\n-- set the HiveCatalog as the current catalog of the session\\nUSE CATALOG hiveCatalog;\\nshow create table customer;\\ncreate external table customer(\\n c_customer_sk bigint,\\n c_customer_id string,\\n c_current_cdemo_sk bigint,\\n c_current_hdemo_sk bigint,\\n c_current_addr_sk bigint,\\n c_first_shipto_date_sk bigint,\\n c_first_sales_date_sk bigint,\\n c_salutation string,\\n c_first_name string,\\n c_last_name string,\\n c_preferred_cust_flag string,\\n c_birth_day int,\\n c_birth_month int,\\n c_birth_year int,\\n c_birth_country string,\\n c_login string,\\n c_email_address string\\n) \\nrow format delimited fields terminated by '|'\\nlocation 'alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/30/customer' \\nTBLPROPERTIES (\\n 'streaming-source.enable' = 'false', \\n 'lookup.join.cache.ttl' = '12 h'\\n)\\n```\n\n如上图所示,可以看到该维度表 location 路径是 alluxio 缓存 ufs 路径的 uri,业务程序读写该维度表时,alluxio 会自动更新缓存中的 customer 维度表数据,并异步写入到 alluxio的backend storage 的 S3表路径,实现数据湖的表数据同步更新。\n\n\n##### **Flink Temporal 时态表 join**\n\n\nFlink 时态表(Temporal table)也是动态表的一种,时态表的每条记录都会有一个或多个时间字段相关联,当我们事实表 join 维度表的时候,通常需要获取实时的维度表数据做 lookup,所以通常需要在事实表 create table 或者 join 时,通过 proctime()函数指定事实表的时间字段,同时在 join 时,通过 FOR SYSTEM_TIME AS OF 语法,指定维度表 lookup 时对应的事实表时间版本的数据\n\n在本 Demo 示例中,客户信息在 hive 离线表作为一个变化的维度表的角色,客户行为在 kafka 中作为事实表的角色,因此在 flink kafka source table 中,通过 proctime()指定时间字段,然后在 flink hive table 做 join 时,使用 FOR SYSTEM_TIME AS OF 指定 lookup 的 kafka source table 的时间字段,从而实现 Flink temporal 时态表 join 业务处理\n\n如下所示,Flink Sql 中通过 Kafka connector 创建用户行为的事实表,其中 ts 字段即为时态表 join 时的时间戳:\n\n```\\nCREATE TABLE logevent_source (`timestamp` string, \\n`system` string,\\n actor STRING,\\n action STRING,\\n ts as PROCTIME()\\n) WITH (\\n'connector' = 'kafka',\\n'topic' = 'logevent',\\n'properties.bootstrap.servers' = 'b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092 (http://b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092/)',\\n'properties.group.id' = 'testGroup-01',\\n'scan.startup.mode'='latest-offset',\\n'format' = 'json'\\n);\\n```\n\nFlink 离线维度表与流式实时表具体 join 方法如下:\n\n ```\\nselect a.`timestamp`,a.`system`,a.actor,a.action,b.c_login from \\n (select *, proctime() as proctime from user_logevent_source) as a \\n left join customer FOR SYSTEM_TIME AS OF a.proctime as b on a.actor=b.c_last_name;\\n```\n\n如上代码示例,在事实表 logevent_source join lookup 维度表时,通过 proctime 函数获取到维度表的瞬时最新的版本数据,保障 join 时的一致性和实时性\n\n同时,该维度表数据已经在 alluxio cache,因此读取时性能远高于离线读取 s3上的表数据\n\n通过 hive 切换 S3和 alluxio 路径的 customer 信息 维度表,对比测试 flink join 可以看出 alluxio 缓存后性能明显优势\n\n通过 alter table 方便切换本地和 cache 的 location路径:\n\n```\\nalter table customer set location \\"s3://xxxxxx/data/s3/30/customer\\";\\nalter table customer set location \\"alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/30/customer\\";\\n```\n\n选取某一 split 数据分片的 TaskManager 日志:\n\n- cache 前(S3路径读取): 5s 加载\n\n```\\n2022-06-29 02:54:34,791 INFO com.amazon.ws.emr.hadoop.fs.s3n.S3NativeFileSystem [] - Opening 's3://salunchbucket/data/s3/30/customer/data-m-00029' for reading\\n2022-06-29 02:54:39,971 INFO org.apache.flink.table.filesystem.FileSystemLookupFunction [] - Loaded 433000 row(s) into lookup join cache\\n```\n\n- cache 后(alluxio 读取): 2s 加载\n\n```\\n2022-06-29 03:25:14,476 INFO com.amazon.ws.emr.hadoop.fs.s3n.S3NativeFileSystem [] - Opening 's3://salunchbucket/data/s3/30/customer/data-m-00029' for reading\\n2022-06-29 03:25:16,397 INFO org.apache.flink.table.filesystem.FileSystemLookupFunction [] - Loaded 433000 row(s) into lookup join cache\\n```\n\n在 JobManager 上查看 Timeline,对比 alluxio 和 s3路径下 job 的执行时间可以看到更加清楚\n\n![image.png](https://dev-media.amazoncloud.cn/e4b6710dd2ee485aacddb14b5d5e5bf6_image.png)\n\n可以看到, 单个 task 查询提升1倍以上,整体 job 性能提升更加明显\n\n\n##### **其他需要考虑的问题**\n\n\n持续 Join 每次都需要拉取维度数据做 join,Flink 的 checkpoint state 是否一直膨胀导致 TM 的 RockDB 撑爆或者内存溢出?\n\nstate 自带有 ttl 机制,可以设置 ttl 过期策略,触发 Flink 清理过期 state 数据,Flink Sql 可以通过 Hint 方式设置\n\n```\\ninsert into logevent_sink\\nselect a.`timestamp`,a.`system`,a.actor,a.action,b.c_login from \\n(select *, proctime() as proctime from logevent_source) as a \\n left join \\ncustomer/*+ OPTIONS('lookup.join.cache.ttl' = '5 min')*/ FOR SYSTEM_TIME AS OF a.proctime as b \\non a.actor=b.c_last_name;\\n```\n\nFlink Table/Streaming API 类似:\n\n```\\nStateTtlConfig ttlConfig = StateTtlConfig\\n .newBuilder(Time.days(7))\\n .setUpdateType(StateTtlConfig.UpdateType.OnCreateAndWrite)\\n .setStateVisibility(StateTtlConfig.StateVisibility.NeverReturnExpired)\\n .cleanupInRocksdbCompactFilter() \\n .build();\\nValueStateDescriptor<Long> lastUserLogin = \\n new ValueStateDescriptor<>(\\"lastUserLogin\\", Long.class);\\nlastUserLogin.enableTimeToLive(ttlConfig);\\nStreamTableEnvironment.getConfig().setIdleStateRetentionTime(min, max);\\n```\n\n设置后重新启动 lookup join,从 Flink TM 日志中可以看到,ttl 到期后,会触发清理并重新拉取 hive 维表数据:\n\n```\\n2022-06-29 04:17:09,161 INFO org.apache.flink.table.filesystem.FileSystemLookupFunction \\n[] - Lookup join cache has expired after 5 minute(s), reloading\\n```\n\n此外,可以通过配置 flink state retain,减少 checkpoint 时候快照数量,从而减少快照时候 state 的占用空间\n\n```\\nFlink job中配置:\\n-D state.checkpoints.num-retained=5\\n```\n \n设置后,可以看到 s3 checkpoint 路径上,Flink Job 会自动清理历史快照,只保留最近的5次快照数据,从而确保 checkpoint 快照数据不会堆积\n\n```\\n[hadoop@ip-172-31-41-131 ~]\$ aws s3 ls s3://salunchbucket/data/checkpoints/7b9f2f9becbf3c879cd1e5f38c6239f8/\\n PRE chk-3/\\n PRE chk-4/\\n PRE chk-5/\\n PRE chk-6/\\n PRE chk-7/\\n```\n\n\n#### **附录**\n\n\n[Alluxio整体架构](https://docs.alluxio.io/ee/user/stable/en/cloud/AWS-EMR.html)\n\n[Alluxio on EMR 快速部署](https://aws.amazon.com/cn/blogs/china/five-minitues-to-use-alluxio-guide-emr-spark/)\n\n[在 Amazon EMR 中利用 Alluxio 的分层存储架构](https://aws.amazon.com/cn/blogs/china/leverage-allusio-tiered-storage-architecture-in-amazon-emr/)\n\n[EMR Alluxio集成detail](https://aws.amazon.com/cn/blogs/china/leverage-allusio-tiered-storage-architecture-in-amazon-emr/)\n\n[Flink Temporal Join 详细](https://nightlies.apache.org/flink/flink-docs-release-1.14/docs/dev/table/sql/queries/joins/)\n\n\n#### **本篇作者**\n\n\n![image.png](https://dev-media.amazoncloud.cn/251089bc94a3449988ff3e039457bd75_image.png)\n\n\n#### **唐清原**\n\n\nAmazon 数据分析解决方案架构师,负责 Amazon Data Analytic 服务方案架构设计以及性能优化,迁移,治理等 Deep Dive 支持。10+数据领域研发及架构设计经验,历任 Oracle 高级咨询顾问,咪咕文化数据集市高级架构师,澳新银行数据分析领域架构师职务。在大数据,数据湖,智能湖仓,及相关推荐系统 /MLOps 平台等项目有丰富实战经验\n\n![image.png](https://dev-media.amazoncloud.cn/77657e1cfaf649ea965a2f278047e707_image.png)\n\n\n#### **陈昊**\n\n\nAmazon 合作伙伴解决方案架构师,有将近 20 年的 IT 从业经验,在企业应用开发、架构设计及建设方面具有丰富的实践经验。目前主要负责 Amazon (中国)合作伙伴的方案架构咨询和设计工作,致力于 Amazon 云服务在国内的应用推广以及帮助合作伙伴构建更高效的 Amazon 云服务解决方案。","render":"<h4><a id=\\"_2\\"></a><strong>业务背景&amp;痛点</strong></h4>\\n<ul>\\n<li>流式处理的业务场景,经常会遇到实时消息数据需要与历史存量数据关联查询或者聚合,比如电商常见的订单场景,订单表做为实时事实表,是典型的流式消息数据,通常会在 kafka 中,而客户信息,商品 SKU 表是维度表,通常存在业务数据库或者数仓中,是典型的离线数据。实时订单数据在实时处理时通常需要事实表与维度表 join 做 reference 补全,以便拿到订单详情并实时统计当天或截至当天的所有订单的商品分布详情。</li>\n<li>流式计算通常采用 Flink 做为数据处理平台,上文中提到的实时和离线数据join 的场景,Flink 提供了 Hive/ jdbc/Hudi/ filesystem 各种 connector 实现与离线数据的提取和读写,这样一来在 Flink 应用程序中,即可使用 Table,Sql API 来 join 关联流态表和离线表数据,实现聚合计算等操作</li>\n</ul>\\n<p>使用 Flink Sql 离线表 Join 流态表的常规 lookup join,是通过 Flink hive sql connector 或者 filesystem connector,对离线 hive 库表或者 S3上离线数据建 Flink Table,然后对 kafka 消息流中的数据建流态表,然后直接做量表做 join 操作</p>\n<p>该方式架构如下图所示:</p>\n<p><img src=\\"https://dev-media.amazoncloud.cn/18381b288b3f4762a86704a4afd2cea1_image.png\\" alt=\\"image.png\\" /></p>\n<p>该方式主要面临的问题是:</p>\n<ul>\\n<li>lookup 维度表数据只会在首次拉起 Flink 应用的时候,保存在 task manager state 中,后续持续查询或者开窗聚合等操作时,是不会再次拉取维度表数据,业务需要定期重启 Flink 应用,或者刷新维度表数据到临时表,以便 join 聚合时和最新的维度数据关联:</li>\n</ul>\\n<p><img src=\\"https://dev-media.amazoncloud.cn/c75ea2eb525f4560bc5eaa5bd16729ac_image.png\\" alt=\\"image.png\\" /></p>\n<ul>\\n<li>每次需要重新全量拉取维度表数据,存在冷启动问题,且维度表数据量大的时候(如上千万注册用户信息表,上万的商品 SKU 属性字段),造成很大 IO 开销,存在性能瓶颈</li>\n<li>Flink 的 checkpoint 机制在持续查询或者开窗聚合时,需要保存 state 状态及处理数据到检查点快照中,造成 state 快照数据膨胀</li>\n</ul>\\n<h4><a id=\\"_24\\"></a><strong>解决方案思路</strong></h4>\\n<p>基于以上业务难点,本文提出一种解决方案思路,即通过 Alluxio 缓存层,将 hive 维度表数据自动加载至 Alluxio UFS 缓存中,同时通过 Flink 时态表 join,把维度表数据做成持续变化表上某一时刻的视图</p>\n<p>同时使用 Flink 的 Temporal table function 表函数,传递一个时间参数,返回 Temporal table 这一指定时刻的视图,这样实时动态表主表与这个 Temporal table 表关联的时候,可以关联到某一个版本(历史上某一个时刻)的维度数据</p>\n<p>优化后的整体架构如下图所示:</p>\n<p><img src=\\"https://dev-media.amazoncloud.cn/4b84fb4f6b9f4dd08dc1d1117712ecc1_image.png\\" alt=\\"image.png\\" /></p>\n<h4><a id=\\"Detail_36\\"></a><strong>方案实施落地Detail</strong></h4>\\n<p>本文以 Kafka 中用户行为日志数据做为实时流态的事实表数据,hive 上用户信息数据做为离线维度表数据,采用 Alluxio+Flink temproal 的 demo,来验证其 flink join 优化的解决方案</p>\n<h5><a id=\\"_42\\"></a><strong>实时事实表</strong></h5>\\n<p>本实例中我们使用 json-data-generator 开源组件模拟的用户行为 json 数据,实时写入 kafka 中,通过 Flink kafka connector 转换为持续查询的 Flink 流态表,从而做为实时 join 的时候的 Fact 事实表数据</p>\n<p>用户行为 json 模拟数据如下格式:</p>\n<pre><code class=\\"lang-\\">[{ &quot;timestamp&quot;: &quot;nowTimestamp()&quot;,\\n &quot;system&quot;: &quot;BADGE&quot;,\\n &quot;actor&quot;: &quot;Agnew&quot;,\\n &quot;action&quot;: &quot;EXIT&quot;,\\n &quot;objects&quot;: [&quot;Building 1&quot;],\\n &quot;location&quot;: &quot;45.5,44.3&quot;,\\n &quot;message&quot;: &quot;Exited Building 1&quot;\\n }]\\n</code></pre>\\n<p>包含用户行为的业务时间,登录系统,用户署名,行为 activity 动作,操作涉及对象,位置信息,及相关文本消息字段。我们在</p>\n<p>Flink Sql 中建选择主要字段建事实表如下</p>\n<pre><code class=\\"lang-\\">CREATE TABLE logevent_source (`timestamp` string, \\n`system` string,\\n actor STRING,\\n action STRING\\n) WITH (\\n'connector' = 'kafka',\\n'topic' = 'logevent',\\n'properties.bootstrap.servers' = 'b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092 (http://b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092/)',\\n'properties.group.id' = 'testGroup6',\\n'scan.startup.mode'='latest-offset',\\n'format' = 'json'\\n);\\n</code></pre>\\n<h5><a id=\\"Alluxio__80\\"></a><strong>Alluxio 缓存维度表</strong></h5>\\n<p>Alluxio 是大数据技术堆栈的分布式缓存,它提供了一个统一的 UFS 文件系统可以对接底层 S3,hdfs 数据,在读写 Alluxio UFS 的时候,可以针对 S3,HDFS 分布式存储层实现 warm up,显著提升吞吐量和减少网络开销,且与上层计算引擎如 Hive,spark,Trino 都有深度的集成,很适合做为离线维度数据的缓存加速器</p>\n<p>Amazon EMR 对 Alluxio 提供了良好的集成,可以通过 boostrap 启动脚本方式,在 EMR 创建时自动部署 Alluxio 组件并启动 Alluxio master、worker 进程,详细 EMR 安装和部署 Alluxio 步骤可以参考另一篇文章 <a href=\\"https://quip-amazon.com/zddcAHEvMDvW\\" target=\\"_blank\\">Alluxio EMR 集成实践</a></p>\\n<p>在集成 Alluxio 的 Amazon EMR 集群中,使用 Alluxio 中创建 hive 离线维表数据的缓存表方法如下:</p>\n<pre><code class=\\"lang-\\">hive-env.sh中设置设置client jar包:\\n\$ export HIVE_AUX_JARS_PATH=/&lt;PATH_TO_ALLUXIO&gt;/client/alluxio-2.2.0-client.jar:\${HIVE_AU\\n\\n确保安装部署alluxio的EMR集群上ufs已配置,并且表或者db路径已创建\\nalluxio fs mkdir alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/customer\\nalluxio fs chown hadoop:hadoop alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/customer\\n\\n在AWS EMR集群上,创建hive表路径指向alluxio namespace uri:\\n!connect jdbc:hive2://xxx.xxx.xxx.xxx:10000/default;\\nhive&gt; CREATE TABLE customer(\\n c_customer_sk bigint,\\n c_customer_id string,\\n c_current_cdemo_sk bigint,\\n c_current_hdemo_sk bigint,\\n c_current_addr_sk bigint,\\n c_first_shipto_date_sk bigint,\\n c_first_sales_date_sk bigint,\\n c_salutation string,\\n c_first_name string,\\n c_last_name string,\\n c_preferred_cust_flag string,\\n c_birth_day int,\\n c_birth_month int,\\n c_birth_year int,\\n c_birth_country string,\\n c_login string,\\n c_email_address string\\n)\\n ROW FORMAT DELIMITED\\n FIELDS TERMINATED BY '|'\\n STORED AS TEXTFILE\\n LOCATION 'alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/customer';\\nOK\\nTime taken: 3.485 seconds\\n</code></pre>\\n<p>如上所示,该 Alluxio 表 location 指向的路径即为 hive 维度表所在 S3路径,因此对 Customer 用户维度信息表的写入操作会自动同步到 alluxio 缓存中。</p>\n<p>创建好 Alluxio hive 离线维度表后,在 flink sql中,可以通过 hive 的 catalog,连接到 hive 元数据,即可以查看到 alluxio 缓存表的详细信息:</p>\n<pre><code class=\\"lang-\\">CREATE CATALOG hiveCatalog WITH ( 'type' = 'hive',\\n 'default-database' = 'default',\\n 'hive-conf-dir' = '/etc/hive/conf/',\\n 'hive-version' = '3.1.2',\\n 'hadoop-conf-dir'='/etc/hadoop/conf/'\\n);\\n-- set the HiveCatalog as the current catalog of the session\\nUSE CATALOG hiveCatalog;\\nshow create table customer;\\ncreate external table customer(\\n c_customer_sk bigint,\\n c_customer_id string,\\n c_current_cdemo_sk bigint,\\n c_current_hdemo_sk bigint,\\n c_current_addr_sk bigint,\\n c_first_shipto_date_sk bigint,\\n c_first_sales_date_sk bigint,\\n c_salutation string,\\n c_first_name string,\\n c_last_name string,\\n c_preferred_cust_flag string,\\n c_birth_day int,\\n c_birth_month int,\\n c_birth_year int,\\n c_birth_country string,\\n c_login string,\\n c_email_address string\\n) \\nrow format delimited fields terminated by '|'\\nlocation 'alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/30/customer' \\nTBLPROPERTIES (\\n 'streaming-source.enable' = 'false', \\n 'lookup.join.cache.ttl' = '12 h'\\n)\\n</code></pre>\\n<p>如上图所示,可以看到该维度表 location 路径是 alluxio 缓存 ufs 路径的 uri,业务程序读写该维度表时,alluxio 会自动更新缓存中的 customer 维度表数据,并异步写入到 alluxio的backend storage 的 S3表路径,实现数据湖的表数据同步更新。</p>\n<h5><a id=\\"Flink_Temporal__join_170\\"></a><strong>Flink Temporal 时态表 join</strong></h5>\\n<p>Flink 时态表(Temporal table)也是动态表的一种,时态表的每条记录都会有一个或多个时间字段相关联,当我们事实表 join 维度表的时候,通常需要获取实时的维度表数据做 lookup,所以通常需要在事实表 create table 或者 join 时,通过 proctime()函数指定事实表的时间字段,同时在 join 时,通过 FOR SYSTEM_TIME AS OF 语法,指定维度表 lookup 时对应的事实表时间版本的数据</p>\n<p>在本 Demo 示例中,客户信息在 hive 离线表作为一个变化的维度表的角色,客户行为在 kafka 中作为事实表的角色,因此在 flink kafka source table 中,通过 proctime()指定时间字段,然后在 flink hive table 做 join 时,使用 FOR SYSTEM_TIME AS OF 指定 lookup 的 kafka source table 的时间字段,从而实现 Flink temporal 时态表 join 业务处理</p>\n<p>如下所示,Flink Sql 中通过 Kafka connector 创建用户行为的事实表,其中 ts 字段即为时态表 join 时的时间戳:</p>\n<pre><code class=\\"lang-\\">CREATE TABLE logevent_source (`timestamp` string, \\n`system` string,\\n actor STRING,\\n action STRING,\\n ts as PROCTIME()\\n) WITH (\\n'connector' = 'kafka',\\n'topic' = 'logevent',\\n'properties.bootstrap.servers' = 'b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092,b-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092 (http://b-6.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-5.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092%2Cb-1.msk06.dr04w4.c3.kafka.ap-southeast-1.amazonaws.com:9092/)',\\n'properties.group.id' = 'testGroup-01',\\n'scan.startup.mode'='latest-offset',\\n'format' = 'json'\\n);\\n</code></pre>\\n<p>Flink 离线维度表与流式实时表具体 join 方法如下:</p>\n<pre><code class=\\"lang-\\">select a.`timestamp`,a.`system`,a.actor,a.action,b.c_login from \\n (select *, proctime() as proctime from user_logevent_source) as a \\nleft join customer FOR SYSTEM_TIME AS OF a.proctime as b on a.actor=b.c_last_name;\\n</code></pre>\\n<p>如上代码示例,在事实表 logevent_source join lookup 维度表时,通过 proctime 函数获取到维度表的瞬时最新的版本数据,保障 join 时的一致性和实时性</p>\n<p>同时,该维度表数据已经在 alluxio cache,因此读取时性能远高于离线读取 s3上的表数据</p>\n<p>通过 hive 切换 S3和 alluxio 路径的 customer 信息 维度表,对比测试 flink join 可以看出 alluxio 缓存后性能明显优势</p>\n<p>通过 alter table 方便切换本地和 cache 的 location路径:</p>\n<pre><code class=\\"lang-\\">alter table customer set location &quot;s3://xxxxxx/data/s3/30/customer&quot;;\\nalter table customer set location &quot;alluxio://ip-xxx-xxx-xxx-xxx.ap-southeast-1.compute.internal:19998/s3/30/customer&quot;;\\n</code></pre>\\n<p>选取某一 split 数据分片的 TaskManager 日志:</p>\n<ul>\\n<li>cache 前(S3路径读取): 5s 加载</li>\n</ul>\\n<pre><code class=\\"lang-\\">2022-06-29 02:54:34,791 INFO com.amazon.ws.emr.hadoop.fs.s3n.S3NativeFileSystem [] - Opening 's3://salunchbucket/data/s3/30/customer/data-m-00029' for reading\\n2022-06-29 02:54:39,971 INFO org.apache.flink.table.filesystem.FileSystemLookupFunction [] - Loaded 433000 row(s) into lookup join cache\\n</code></pre>\\n<ul>\\n<li>cache 后(alluxio 读取): 2s 加载</li>\n</ul>\\n<pre><code class=\\"lang-\\">2022-06-29 03:25:14,476 INFO com.amazon.ws.emr.hadoop.fs.s3n.S3NativeFileSystem [] - Opening 's3://salunchbucket/data/s3/30/customer/data-m-00029' for reading\\n2022-06-29 03:25:16,397 INFO org.apache.flink.table.filesystem.FileSystemLookupFunction [] - Loaded 433000 row(s) into lookup join cache\\n</code></pre>\\n<p>在 JobManager 上查看 Timeline,对比 alluxio 和 s3路径下 job 的执行时间可以看到更加清楚</p>\n<p><img src=\\"https://dev-media.amazoncloud.cn/e4b6710dd2ee485aacddb14b5d5e5bf6_image.png\\" alt=\\"image.png\\" /></p>\n<p>可以看到, 单个 task 查询提升1倍以上,整体 job 性能提升更加明显</p>\n<h5><a id=\\"_239\\"></a><strong>其他需要考虑的问题</strong></h5>\\n<p>持续 Join 每次都需要拉取维度数据做 join,Flink 的 checkpoint state 是否一直膨胀导致 TM 的 RockDB 撑爆或者内存溢出?</p>\n<p>state 自带有 ttl 机制,可以设置 ttl 过期策略,触发 Flink 清理过期 state 数据,Flink Sql 可以通过 Hint 方式设置</p>\n<pre><code class=\\"lang-\\">insert into logevent_sink\\nselect a.`timestamp`,a.`system`,a.actor,a.action,b.c_login from \\n(select *, proctime() as proctime from logevent_source) as a \\n left join \\ncustomer/*+ OPTIONS('lookup.join.cache.ttl' = '5 min')*/ FOR SYSTEM_TIME AS OF a.proctime as b \\non a.actor=b.c_last_name;\\n</code></pre>\\n<p>Flink Table/Streaming API 类似:</p>\n<pre><code class=\\"lang-\\">StateTtlConfig ttlConfig = StateTtlConfig\\n .newBuilder(Time.days(7))\\n .setUpdateType(StateTtlConfig.UpdateType.OnCreateAndWrite)\\n .setStateVisibility(StateTtlConfig.StateVisibility.NeverReturnExpired)\\n .cleanupInRocksdbCompactFilter() \\n .build();\\nValueStateDescriptor&lt;Long&gt; lastUserLogin = \\n new ValueStateDescriptor&lt;&gt;(&quot;lastUserLogin&quot;, Long.class);\\nlastUserLogin.enableTimeToLive(ttlConfig);\\nStreamTableEnvironment.getConfig().setIdleStateRetentionTime(min, max);\\n</code></pre>\\n<p>设置后重新启动 lookup join,从 Flink TM 日志中可以看到,ttl 到期后,会触发清理并重新拉取 hive 维表数据:</p>\n<pre><code class=\\"lang-\\">2022-06-29 04:17:09,161 INFO org.apache.flink.table.filesystem.FileSystemLookupFunction \\n[] - Lookup join cache has expired after 5 minute(s), reloading\\n</code></pre>\\n<p>此外,可以通过配置 flink state retain,减少 checkpoint 时候快照数量,从而减少快照时候 state 的占用空间</p>\n<pre><code class=\\"lang-\\">Flink job中配置:\\n-D state.checkpoints.num-retained=5\\n</code></pre>\\n<p>设置后,可以看到 s3 checkpoint 路径上,Flink Job 会自动清理历史快照,只保留最近的5次快照数据,从而确保 checkpoint 快照数据不会堆积</p>\n<pre><code class=\\"lang-\\">[hadoop@ip-172-31-41-131 ~]\$ aws s3 ls s3://salunchbucket/data/checkpoints/7b9f2f9becbf3c879cd1e5f38c6239f8/\\n PRE chk-3/\\n PRE chk-4/\\n PRE chk-5/\\n PRE chk-6/\\n PRE chk-7/\\n</code></pre>\\n<h4><a id=\\"_296\\"></a><strong>附录</strong></h4>\\n<p><a href=\\"https://docs.alluxio.io/ee/user/stable/en/cloud/AWS-EMR.html\\" target=\\"_blank\\">Alluxio整体架构</a></p>\\n<p><a href=\\"https://aws.amazon.com/cn/blogs/china/five-minitues-to-use-alluxio-guide-emr-spark/\\" target=\\"_blank\\">Alluxio on EMR 快速部署</a></p>\\n<p><a href=\\"https://aws.amazon.com/cn/blogs/china/leverage-allusio-tiered-storage-architecture-in-amazon-emr/\\" target=\\"_blank\\">在 Amazon EMR 中利用 Alluxio 的分层存储架构</a></p>\\n<p><a href=\\"https://aws.amazon.com/cn/blogs/china/leverage-allusio-tiered-storage-architecture-in-amazon-emr/\\" target=\\"_blank\\">EMR Alluxio集成detail</a></p>\\n<p><a href=\\"https://nightlies.apache.org/flink/flink-docs-release-1.14/docs/dev/table/sql/queries/joins/\\" target=\\"_blank\\">Flink Temporal Join 详细</a></p>\\n<h4><a id=\\"_310\\"></a><strong>本篇作者</strong></h4>\\n<p><img src=\\"https://dev-media.amazoncloud.cn/251089bc94a3449988ff3e039457bd75_image.png\\" alt=\\"image.png\\" /></p>\n<h4><a id=\\"_316\\"></a><strong>唐清原</strong></h4>\\n<p>Amazon 数据分析解决方案架构师,负责 Amazon Data Analytic 服务方案架构设计以及性能优化,迁移,治理等 Deep Dive 支持。10+数据领域研发及架构设计经验,历任 Oracle 高级咨询顾问,咪咕文化数据集市高级架构师,澳新银行数据分析领域架构师职务。在大数据,数据湖,智能湖仓,及相关推荐系统 /MLOps 平台等项目有丰富实战经验</p>\n<p><img src=\\"https://dev-media.amazoncloud.cn/77657e1cfaf649ea965a2f278047e707_image.png\\" alt=\\"image.png\\" /></p>\n<h4><a id=\\"_324\\"></a><strong>陈昊</strong></h4>\\n<p>Amazon 合作伙伴解决方案架构师,有将近 20 年的 IT 从业经验,在企业应用开发、架构设计及建设方面具有丰富的实践经验。目前主要负责 Amazon (中国)合作伙伴的方案架构咨询和设计工作,致力于 Amazon 云服务在国内的应用推广以及帮助合作伙伴构建更高效的 Amazon 云服务解决方案。</p>\n"}
目录
亚马逊云科技解决方案 基于行业客户应用场景及技术领域的解决方案
联系亚马逊云科技专家
亚马逊云科技解决方案
基于行业客户应用场景及技术领域的解决方案
联系专家
0
目录
关闭