大数据-131 - Flink CEP 案例:检测交易活跃用户、超时未交付

简介: 大数据-131 - Flink CEP 案例:检测交易活跃用户、超时未交付

点一下关注吧!!!非常感谢!!持续更新!!!

目前已经更新到了:

Hadoop(已更完)

HDFS(已更完)

MapReduce(已更完)

Hive(已更完)

Flume(已更完)

Sqoop(已更完)

Zookeeper(已更完)

HBase(已更完)

Redis (已更完)

Kafka(已更完)

Spark(已更完)

Flink(正在更新!)

章节内容

上节我们完成了如下的内容:


Flink CEP 开发的流程

CEP 开发依赖

CEP 案例:恶意登录检测实现

Fline CEP

之前已经介绍过,但是防止大家没看到,这里再简单介绍以下。


基本概念

Flink CEP(Complex Event Processing)是Apache Flink提供的一个扩展库,用于实时复杂事件处理。通过Flink CEP,开发者可以从流数据中识别出特定的事件模式。这在欺诈检测、网络安全、实时监控、物联网等场景中非常有用。


Flink CEP的核心是通过定义事件模式,从流中检测复杂事件序列。

具体来说,CEP允许用户:


定义事件模式:用户可以描述感兴趣的事件组合(如连续事件、延迟事件等)。

匹配模式:Flink CEP从流中搜索与定义模式相匹配的事件序列。

处理匹配结果:一旦找到符合模式的事件序列,用户可以定义如何处理这些匹配。

基本组成部分

Pattern(模式):描述要在事件流中匹配的事件序列。可以是单个事件或多个事件的组合。常用的模式操作包括next(紧邻)、followedBy(接续)等。

PatternStream(模式流):通过应用模式定义,将事件流转变为模式流。

Select函数:用于从模式流中提取匹配的事件序列

CEP开发步骤

开发Flink CEP应用的基本步骤包括:


定义事件流:创建一个DataStream,表示原始的事件流。

定义事件模式:使用Flink CEP的API定义事件模式,例如连续事件、迟到事件等。

将模式应用到流中:将定义好的模式应用到事件流上,生成模式流PatternStream。

提取匹配事件:使用select函数提取匹配模式的事件,并定义如何处理这些事件。


使用场景

欺诈检测:可以通过CEP识别连续发生的异常行为,如频繁的登录尝试等。

网络监控:检测一段时间内的特定网络攻击模式。

物联网:分析传感器数据,检测设备异常、温度异常等。

用户行为分析:分析用户在某一时间段内的行为序列,从而作出预测或检测异常。

案例2:检测交易活跃用户

业务需求

业务上需要找出24小时内,至少5次有效交易的用户。

数据源如下:

new CepActiveUserBean("100XX", 0.0D, 1597905234000L),
new CepActiveUserBean("100XX", 100.0D, 1597905235000L),
new CepActiveUserBean("100XX", 200.0D, 1597905236000L),
new CepActiveUserBean("100XX", 300.0D, 1597905237000L),
new CepActiveUserBean("100XX", 400.0D, 1597905238000L),
new CepActiveUserBean("100XX", 500.0D, 1597905239000L),
new CepActiveUserBean("101XX", 0.0D, 1597905240000L),
new CepActiveUserBean("101XX", 100.0D, 1597905241000L)
  • 获取数据源
  • Watermark转化
  • keyBy转化
  • 至少5次:timeOrMore(5)
  • 24小时之内:within(Time.hours(24))
  • 模式匹配
  • 提取匹配成功的数据

编写代码

package icu.wzk;

import org.apache.flink.api.common.eventtime.*;
import org.apache.flink.api.java.functions.KeySelector;
import org.apache.flink.cep.CEP;
import org.apache.flink.cep.PatternStream;
import org.apache.flink.cep.functions.PatternProcessFunction;
import org.apache.flink.cep.pattern.Pattern;
import org.apache.flink.cep.pattern.conditions.SimpleCondition;
import org.apache.flink.streaming.api.TimeCharacteristic;
import org.apache.flink.streaming.api.datastream.DataStreamSource;
import org.apache.flink.streaming.api.datastream.KeyedStream;
import org.apache.flink.streaming.api.datastream.SingleOutputStreamOperator;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.util.Collector;

import java.util.List;
import java.util.Map;


public class FlinkCepActiveUser {

    public static void main(String[] args) throws Exception {
        StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
        env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime);
        env.setParallelism(1);
        DataStreamSource<CepActiveUserBean> data = env.fromElements(
                new CepActiveUserBean("100XX", 0.0D, 1597905234000L),
                new CepActiveUserBean("100XX", 100.0D, 1597905235000L),
                new CepActiveUserBean("100XX", 200.0D, 1597905236000L),
                new CepActiveUserBean("100XX", 300.0D, 1597905237000L),
                new CepActiveUserBean("100XX", 400.0D, 1597905238000L),
                new CepActiveUserBean("100XX", 500.0D, 1597905239000L),
                new CepActiveUserBean("101XX", 0.0D, 1597905240000L),
                new CepActiveUserBean("101XX", 100.0D, 1597905241000L)
        );
        SingleOutputStreamOperator<CepActiveUserBean> watermark = data
                .assignTimestampsAndWatermarks(new WatermarkStrategy<CepActiveUserBean>() {
                    @Override
                    public WatermarkGenerator<CepActiveUserBean> createWatermarkGenerator(WatermarkGeneratorSupplier.Context context) {
                        return new WatermarkGenerator<CepActiveUserBean>() {

                            long maxTimestamp = Long.MAX_VALUE;
                            long maxOutOfOrderness = 500L;

                            @Override
                            public void onEvent(CepActiveUserBean event, long eventTimestamp, WatermarkOutput output) {
                                maxTimestamp = Math.max(event.getTimestamp(), maxTimestamp);
                            }

                            @Override
                            public void onPeriodicEmit(WatermarkOutput output) {
                                output.emitWatermark(new Watermark(maxTimestamp - maxOutOfOrderness));
                            }
                        };
                    }
                }.withTimestampAssigner((element, recordTimes) -> element.getTimestamp())
                );
        KeyedStream<CepActiveUserBean, String> keyed = watermark
                .keyBy(new KeySelector<CepActiveUserBean, String>() {
                    @Override
                    public String getKey(CepActiveUserBean value) throws Exception {
                        return value.getUsername();
                    }
                });
        Pattern<CepActiveUserBean, CepActiveUserBean> pattern = Pattern
                .<CepActiveUserBean>begin("start")
                .where(new SimpleCondition<CepActiveUserBean>() {
                    @Override
                    public boolean filter(CepActiveUserBean value) throws Exception {
                        return value.getPrice() > 0;
                    }
                })
                .timesOrMore(5)
                .within(Time.hours(24));
        PatternStream<CepActiveUserBean> parentStream = CEP.pattern(keyed, pattern);
        SingleOutputStreamOperator<CepActiveUserBean> process = parentStream
                .process(new PatternProcessFunction<CepActiveUserBean, CepActiveUserBean>() {
                    @Override
                    public void processMatch(Map<String, List<CepActiveUserBean>> map, Context context, Collector<CepActiveUserBean> collector) throws Exception {
                        System.out.println("map: " + map);
                    }
                });
        process.print();
        env.execute("FlinkCepActiveUser");
    }

}


class CepActiveUserBean {
    private String username;
    private Double price;
    private Long timestamp;

    public CepActiveUserBean(String username, Double price, Long timestamp) {
        this.username = username;
        this.price = price;
        this.timestamp = timestamp;
    }

    public String getUsername() {
        return username;
    }

    public void setUsername(String username) {
        this.username = username;
    }

    public Double getPrice() {
        return price;
    }

    public void setPrice(Double price) {
        this.price = price;
    }

    public Long getTimestamp() {
        return timestamp;
    }

    public void setTimestamp(Long timestamp) {
        this.timestamp = timestamp;
    }

    @Override
    public String toString() {
        return "CepActiveUserBean{" +
                "username='" + username + '\'' +
                ", price=" + price +
                ", timestamp=" + timestamp +
                '}';
    }
}

运行结果

map: {start=[CepActiveUserBean{username='100XX', price=100.0, timestamp=1597905235000}, CepActiveUserBean{username='100XX', price=200.0, timestamp=1597905236000}, CepActiveUserBean{username='100XX', price=300.0, timestamp=1597905237000}, CepActiveUserBean{username='100XX', price=400.0, timestamp=1597905238000}, CepActiveUserBean{username='100XX', price=500.0, timestamp=1597905239000}]}

Process finished with exit code 0

运行结果如下图所示:

案例3:超时未支付

业务需求

找出下单后10分钟没有支付的订单,数据源如下:

new TimeOutPayBean(1L, "create", 1597905234000L),
new TimeOutPayBean(1L, "pay", 1597905235000L),
new TimeOutPayBean(2L, "create", 1597905236000L),
new TimeOutPayBean(2L, "pay", 1597905237000L),
new TimeOutPayBean(3L, "create", 1597905239000L)
  • 获取数据源
  • 转 Watermark
  • keyBy 转化
  • 做出 Pattern (下单以后10分钟未支付)
  • 模式匹配
  • 取出匹配成功的数据

编写代码

package icu.wzk;

import org.apache.flink.api.common.eventtime.*;
import org.apache.flink.api.java.functions.KeySelector;
import org.apache.flink.cep.CEP;
import org.apache.flink.cep.PatternSelectFunction;
import org.apache.flink.cep.PatternStream;
import org.apache.flink.cep.PatternTimeoutFunction;
import org.apache.flink.cep.pattern.Pattern;
import org.apache.flink.cep.pattern.conditions.IterativeCondition;
import org.apache.flink.streaming.api.TimeCharacteristic;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.datastream.DataStreamSource;
import org.apache.flink.streaming.api.datastream.KeyedStream;
import org.apache.flink.streaming.api.datastream.SingleOutputStreamOperator;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.util.OutputTag;

import java.util.List;
import java.util.Map;


public class FlinkCepTimeOutPay {

    public static void main(String[] args) throws Exception {
        StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
        env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime);
        env.setParallelism(1);
        DataStreamSource<TimeOutPayBean> data = env.fromElements(
                new TimeOutPayBean(1L, "create", 1597905234000L),
                new TimeOutPayBean(1L, "pay", 1597905235000L),
                new TimeOutPayBean(2L, "create", 1597905236000L),
                new TimeOutPayBean(2L, "pay", 1597905237000L),
                new TimeOutPayBean(3L, "create", 1597905239000L)
        );
        DataStream<TimeOutPayBean> watermark = data
                .assignTimestampsAndWatermarks(new WatermarkStrategy<TimeOutPayBean>() {
                    @Override
                    public WatermarkGenerator<TimeOutPayBean> createWatermarkGenerator(WatermarkGeneratorSupplier.Context context) {
                        return new WatermarkGenerator<TimeOutPayBean>() {

                            long maxTimestamp = Long.MAX_VALUE;
                            long maxOutOfOrderness = 500L;

                            @Override
                            public void onEvent(TimeOutPayBean event, long eventTimestamp, WatermarkOutput output) {
                                maxTimestamp = Math.max(maxTimestamp, event.getTimestamp());
                            }

                            @Override
                            public void onPeriodicEmit(WatermarkOutput output) {
                                output.emitWatermark(new Watermark(maxTimestamp - maxOutOfOrderness));
                            }
                        };
                    }
                }.withTimestampAssigner((element, recordTimestamp) -> element.getTimestamp())
                );
        KeyedStream<TimeOutPayBean, Long> keyedStream = watermark
                .keyBy(new KeySelector<TimeOutPayBean, Long>() {
                    @Override
                    public Long getKey(TimeOutPayBean value) throws Exception {
                        return value.getUserId();
                    }
                });
        // 逻辑处理代码
        OutputTag<TimeOutPayBean> orderTimeoutOutput = new OutputTag<>("orderTimeout") {};
        Pattern<TimeOutPayBean, TimeOutPayBean> pattern = Pattern
                .<TimeOutPayBean>begin("begin")
                .where(new IterativeCondition<TimeOutPayBean>() {
                    @Override
                    public boolean filter(TimeOutPayBean timeOutPayBean, Context<TimeOutPayBean> context) throws Exception {
                        return timeOutPayBean.getOperation().equals("create");
                    }
                })
                .followedBy("pay")
                .where(new IterativeCondition<TimeOutPayBean>() {
                    @Override
                    public boolean filter(TimeOutPayBean timeOutPayBean, Context<TimeOutPayBean> context) throws Exception {
                        return timeOutPayBean.getOperation().equals("pay");
                    }
                })
                .within(Time.seconds(600));
        PatternStream<TimeOutPayBean> patternStream = CEP.pattern(keyedStream, pattern);
        SingleOutputStreamOperator<TimeOutPayBean> result = patternStream
                .select(orderTimeoutOutput, new PatternTimeoutFunction<TimeOutPayBean, TimeOutPayBean>() {
                    @Override
                    public TimeOutPayBean timeout(Map<String, List<TimeOutPayBean>> map, long l) throws Exception {
                        return map.get("begin").get(0);
                    }
                }, new PatternSelectFunction<TimeOutPayBean, TimeOutPayBean>() {
                    @Override
                    public TimeOutPayBean select(Map<String, List<TimeOutPayBean>> map) throws Exception {
                        return map.get("pay").get(0);
                    }
                });

        // 输出结果
        // result.print();
        System.out.println("==============");
        DataStream<TimeOutPayBean> sideOutput = result
                .getSideOutput(orderTimeoutOutput);
        sideOutput.print();

        // 执行
        env.execute("FlinkCepTimeOutPay");
    }

}


class TimeOutPayBean {

    private Long userId;

    private String operation;

    private Long timestamp;

    public TimeOutPayBean(Long userId, String operation, Long timestamp) {
        this.userId = userId;
        this.operation = operation;
        this.timestamp = timestamp;
    }

    public Long getUserId() {
        return userId;
    }

    public void setUserId(Long userId) {
        this.userId = userId;
    }

    public String getOperation() {
        return operation;
    }

    public void setOperation(String operation) {
        this.operation = operation;
    }

    public Long getTimestamp() {
        return timestamp;
    }

    public void setTimestamp(Long timestamp) {
        this.timestamp = timestamp;
    }

    @Override
    public String toString() {
        return "TimeOutPayBean{" +
                "userId=" + userId +
                ", operation='" + operation + '\'' +
                ", timestamp=" + timestamp +
                '}';
    }
}

运行结果

控制台输出为:

==============
TimeOutPayBean{userId=1, operation='pay', timestamp=1597905235000}
TimeOutPayBean{userId=3, operation='create', timestamp=1597905239000}
TimeOutPayBean{userId=2, operation='pay', timestamp=1597905237000}

Process finished with exit code 0

对应截图如下:

相关文章
|
27天前
|
弹性计算 人工智能 架构师
阿里云携手Altair共拓云上工业仿真新机遇
2024年9月12日,「2024 Altair 技术大会杭州站」成功召开,阿里云弹性计算产品运营与生态负责人何川,与Altair中国技术总监赵阳在会上联合发布了最新的“云上CAE一体机”。
阿里云携手Altair共拓云上工业仿真新机遇
|
3天前
|
人工智能 Rust Java
10月更文挑战赛火热启动,坚持热爱坚持创作!
开发者社区10月更文挑战,寻找热爱技术内容创作的你,欢迎来创作!
372 16
|
19天前
|
存储 关系型数据库 分布式数据库
GraphRAG:基于PolarDB+通义千问+LangChain的知识图谱+大模型最佳实践
本文介绍了如何使用PolarDB、通义千问和LangChain搭建GraphRAG系统,结合知识图谱和向量检索提升问答质量。通过实例展示了单独使用向量检索和图检索的局限性,并通过图+向量联合搜索增强了问答准确性。PolarDB支持AGE图引擎和pgvector插件,实现图数据和向量数据的统一存储与检索,提升了RAG系统的性能和效果。
|
6天前
|
JSON 自然语言处理 数据管理
阿里云百炼产品月刊【2024年9月】
阿里云百炼产品月刊【2024年9月】,涵盖本月产品和功能发布、活动,应用实践等内容,帮助您快速了解阿里云百炼产品的最新动态。
阿里云百炼产品月刊【2024年9月】
|
21天前
|
人工智能 IDE 程序员
期盼已久!通义灵码 AI 程序员开启邀测,全流程开发仅用几分钟
在云栖大会上,阿里云云原生应用平台负责人丁宇宣布,「通义灵码」完成全面升级,并正式发布 AI 程序员。
|
23天前
|
机器学习/深度学习 算法 大数据
【BetterBench博士】2024 “华为杯”第二十一届中国研究生数学建模竞赛 选题分析
2024“华为杯”数学建模竞赛,对ABCDEF每个题进行详细的分析,涵盖风电场功率优化、WLAN网络吞吐量、磁性元件损耗建模、地理环境问题、高速公路应急车道启用和X射线脉冲星建模等多领域问题,解析了问题类型、专业和技能的需要。
2594 22
【BetterBench博士】2024 “华为杯”第二十一届中国研究生数学建模竞赛 选题分析
|
5天前
|
存储 人工智能 搜索推荐
数据治理,是时候打破刻板印象了
瓴羊智能数据建设与治理产品Datapin全面升级,可演进扩展的数据架构体系为企业数据治理预留发展空间,推出敏捷版用以解决企业数据量不大但需构建数据的场景问题,基于大模型打造的DataAgent更是为企业用好数据资产提供了便利。
182 2
|
3天前
|
编译器 C#
C#多态概述:通过继承实现的不同对象调用相同的方法,表现出不同的行为
C#多态概述:通过继承实现的不同对象调用相同的方法,表现出不同的行为
105 65
|
7天前
|
Linux 虚拟化 开发者
一键将CentOs的yum源更换为国内阿里yum源
一键将CentOs的yum源更换为国内阿里yum源
333 2
|
23天前
|
机器学习/深度学习 算法 数据可视化
【BetterBench博士】2024年中国研究生数学建模竞赛 C题:数据驱动下磁性元件的磁芯损耗建模 问题分析、数学模型、python 代码
2024年中国研究生数学建模竞赛C题聚焦磁性元件磁芯损耗建模。题目背景介绍了电能变换技术的发展与应用,强调磁性元件在功率变换器中的重要性。磁芯损耗受多种因素影响,现有模型难以精确预测。题目要求通过数据分析建立高精度磁芯损耗模型。具体任务包括励磁波形分类、修正斯坦麦茨方程、分析影响因素、构建预测模型及优化设计条件。涉及数据预处理、特征提取、机器学习及优化算法等技术。适合电气、材料、计算机等多个专业学生参与。
1580 17
【BetterBench博士】2024年中国研究生数学建模竞赛 C题:数据驱动下磁性元件的磁芯损耗建模 问题分析、数学模型、python 代码