Commit 2fb0f6f1 authored by 赵建伟's avatar 赵建伟

update codes

parent a9e7cbb1
...@@ -36,7 +36,7 @@ public class PortraitMonitorMain { ...@@ -36,7 +36,7 @@ public class PortraitMonitorMain {
String portraitErrGroupId = parameterTool.get("portraitErrGroupId","flink_monitor_err"); String portraitErrGroupId = parameterTool.get("portraitErrGroupId","flink_monitor_err");
String portraitShdGroupId = parameterTool.get("portraitShdGroupId","flink_monitor_shd"); String portraitShdGroupId = parameterTool.get("portraitShdGroupId","flink_monitor_shd");
String portraitSucGroupId = parameterTool.get("portraitSucGroupId","flink_monitor_suc"); String portraitSucGroupId = parameterTool.get("portraitSucGroupId","flink_monitor_suc");
Integer windowSize = parameterTool.getInt("windowSize",3); Integer windowSize = parameterTool.getInt("windowSize",60000);
Integer slideSize = parameterTool.getInt("slideSize",3); Integer slideSize = parameterTool.getInt("slideSize",3);
String outJdbcUrl = parameterTool.get("outJdbcUrl", String outJdbcUrl = parameterTool.get("outJdbcUrl",
"jdbc:mysql://test002:3306/flink_monitor?user=root&password=5OqYM^zLwotJ3oSo&autoReconnect=true&useSSL=false"); "jdbc:mysql://test002:3306/flink_monitor?user=root&password=5OqYM^zLwotJ3oSo&autoReconnect=true&useSSL=false");
...@@ -49,13 +49,14 @@ public class PortraitMonitorMain { ...@@ -49,13 +49,14 @@ public class PortraitMonitorMain {
// 获得流处理环境对象 // 获得流处理环境对象
StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment(); StreamExecutionEnvironment env = StreamExecutionEnvironment.getExecutionEnvironment();
env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime);
env.enableCheckpointing(1000); env.enableCheckpointing(1000);
env.setStateBackend(new FsStateBackend(checkpointPath)); env.setStateBackend(new FsStateBackend(checkpointPath));
env.setRestartStrategy(RestartStrategies.fixedDelayRestart(1, 3000)); env.setRestartStrategy(RestartStrategies.fixedDelayRestart(1, 3000));
CheckpointConfig config = env.getCheckpointConfig(); CheckpointConfig config = env.getCheckpointConfig();
config.enableExternalizedCheckpoints(CheckpointConfig.ExternalizedCheckpointCleanup.RETAIN_ON_CANCELLATION); config.enableExternalizedCheckpoints(CheckpointConfig.ExternalizedCheckpointCleanup.RETAIN_ON_CANCELLATION);
// env.setStreamTimeCharacteristic(TimeCharacteristic.EventTime);
// 获取数据源 // 获取数据源
DataStream portraitErrDataStream = new PortraitKafkaSource( DataStream portraitErrDataStream = new PortraitKafkaSource(
......
...@@ -8,8 +8,11 @@ import org.apache.flink.api.common.functions.FilterFunction; ...@@ -8,8 +8,11 @@ import org.apache.flink.api.common.functions.FilterFunction;
import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.streaming.api.datastream.DataStream; import org.apache.flink.streaming.api.datastream.DataStream;
import org.joda.time.format.DateTimeFormatter;
import org.joda.time.format.ISODateTimeFormat;
import java.util.Arrays; import java.util.Arrays;
import java.util.Date;
/** /**
* @ClassName PortraitMonitorErr * @ClassName PortraitMonitorErr
...@@ -24,6 +27,8 @@ public class PortraitMonitorErrOperator implements BaseOperator{ ...@@ -24,6 +27,8 @@ public class PortraitMonitorErrOperator implements BaseOperator{
private int maxRetry; private int maxRetry;
private long retryInteral; private long retryInteral;
private int parallelism; private int parallelism;
public static final DateTimeFormatter dateTimeFormat = ISODateTimeFormat.dateTime();
public static final DateTimeFormatter dateTimeNoMillisFormat = ISODateTimeFormat.dateTimeNoMillis();
public PortraitMonitorErrOperator(DataStream dataStream, String outJdbcUrl, int maxRetry, long retryInteral,int parallelism) { public PortraitMonitorErrOperator(DataStream dataStream, String outJdbcUrl, int maxRetry, long retryInteral,int parallelism) {
this.dataStream = dataStream; this.dataStream = dataStream;
...@@ -104,8 +109,26 @@ public class PortraitMonitorErrOperator implements BaseOperator{ ...@@ -104,8 +109,26 @@ public class PortraitMonitorErrOperator implements BaseOperator{
.map(new MapFunction<String,Tuple2<String,String>>() { .map(new MapFunction<String,Tuple2<String,String>>() {
@Override @Override
public Tuple2<String,String> map(String value) throws Exception { public Tuple2<String,String> map(String value) throws Exception {
String monitorTime = DateUtils.getCurrentTimeStr(); long logTime = 0;
return new Tuple2<>(value,monitorTime); JSONObject jsonObject = JSONObject.parseObject(value);
String maidianEventTime = jsonObject.getString("create_at");
if(StringUtils.isNotBlank(maidianEventTime)){
logTime = Long.valueOf(maidianEventTime);
}
String backendEventTime = jsonObject.getString("TIME");
if(StringUtils.isNotBlank(backendEventTime)){
try {
logTime = dateTimeFormat.parseMillis(backendEventTime)/1000;
} catch (IllegalArgumentException e) {
try {
logTime = dateTimeNoMillisFormat.parseMillis(backendEventTime)/1000;
} catch (IllegalArgumentException e2) {
e2.printStackTrace();
}
}
}
String eventTime = DateUtils.secondToDate(logTime);
return new Tuple2<>(value,eventTime);
} }
}) })
.addSink(new PortraitErrMysqlSink(outJdbcUrl,maxRetry,retryInteral)) .addSink(new PortraitErrMysqlSink(outJdbcUrl,maxRetry,retryInteral))
......
package com.gmei.data.monitor.operator; package com.gmei.data.monitor.operator;
import com.alibaba.fastjson.JSONObject; import com.alibaba.fastjson.JSONObject;
import com.gmei.data.monitor.bean.GmPortraitResult;
import com.gmei.data.monitor.bean.TblMonitorPortraitShd; import com.gmei.data.monitor.bean.TblMonitorPortraitShd;
import com.gmei.data.monitor.sink.PortraitShdMysqlSink; import com.gmei.data.monitor.sink.PortraitShdMysqlSink;
import com.gmei.data.monitor.utils.DateUtils; import com.gmei.data.monitor.utils.DateUtils;
import org.apache.commons.lang3.StringUtils;
import org.apache.flink.api.common.functions.FilterFunction; import org.apache.flink.api.common.functions.FilterFunction;
import org.apache.flink.api.common.functions.MapFunction; import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.java.tuple.Tuple; import org.apache.flink.api.java.tuple.Tuple;
import org.apache.flink.api.java.tuple.Tuple2; import org.apache.flink.api.java.tuple.Tuple2;
import org.apache.flink.streaming.api.datastream.DataStream; import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.datastream.KeyedStream;
import org.apache.flink.streaming.api.datastream.SingleOutputStreamOperator; import org.apache.flink.streaming.api.datastream.SingleOutputStreamOperator;
import org.apache.flink.streaming.api.datastream.WindowedStream;
import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment; import org.apache.flink.streaming.api.environment.StreamExecutionEnvironment;
import org.apache.flink.streaming.api.functions.windowing.ProcessWindowFunction; import org.apache.flink.streaming.api.functions.windowing.ProcessWindowFunction;
import org.apache.flink.streaming.api.windowing.time.Time; import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.streaming.api.windowing.windows.TimeWindow; import org.apache.flink.streaming.api.windowing.windows.TimeWindow;
import org.apache.flink.util.Collector; import org.apache.flink.util.Collector;
import org.joda.time.format.DateTimeFormatter;
import org.joda.time.format.ISODateTimeFormat;
import java.util.Arrays; import java.util.Arrays;
import java.util.Date;
/** /**
* @ClassName PortraitMonitorShdOperator * @ClassName PortraitMonitorShdOperator
...@@ -33,6 +40,8 @@ public class PortraitMonitorShdOperator implements BaseOperator{ ...@@ -33,6 +40,8 @@ public class PortraitMonitorShdOperator implements BaseOperator{
private int maxRetry; private int maxRetry;
private long retryInteral; private long retryInteral;
private int parallelism; private int parallelism;
public static final DateTimeFormatter dateTimeFormat = ISODateTimeFormat.dateTime();
public static final DateTimeFormatter dateTimeNoMillisFormat = ISODateTimeFormat.dateTimeNoMillis();
public PortraitMonitorShdOperator(DataStream dataStream, int windownSize,int slideSize,String outJdbcUrl, int maxRetry, long retryInteral,int parallelism) { public PortraitMonitorShdOperator(DataStream dataStream, int windownSize,int slideSize,String outJdbcUrl, int maxRetry, long retryInteral,int parallelism) {
this.dataStream = dataStream; this.dataStream = dataStream;
...@@ -46,102 +55,130 @@ public class PortraitMonitorShdOperator implements BaseOperator{ ...@@ -46,102 +55,130 @@ public class PortraitMonitorShdOperator implements BaseOperator{
@Override @Override
public void run() { public void run() {
dataStream SingleOutputStreamOperator map01 = dataStream.map(new MapFunction<String, JSONObject>() {
.map(new MapFunction<String, JSONObject>() { @Override
@Override public JSONObject map(String value) throws Exception {
public JSONObject map(String value) throws Exception { JSONObject jsonObject = new JSONObject();
JSONObject jsonObject = new JSONObject(); try {
try{ jsonObject = JSONObject.parseObject(value);
jsonObject = JSONObject.parseObject(value); } catch (Exception e) {
}catch (Exception e){ e.printStackTrace();
e.printStackTrace();
}
return jsonObject;
} }
}) return jsonObject;
.filter(new FilterFunction<JSONObject>() { }
@Override });
public boolean filter(JSONObject jsonObject) throws Exception { //map01.print();
try { SingleOutputStreamOperator filter01 = map01.filter(new FilterFunction<JSONObject>() {
JSONObject sysObject = jsonObject.getJSONObject("SYS"); @Override
if (null != sysObject) { public boolean filter(JSONObject jsonObject) throws Exception {
String action = sysObject.getString("action"); try {
if (null != action) { JSONObject sysObject = jsonObject.getJSONObject("SYS");
if ("/api/private_conversation/".equals(action) if (null != sysObject) {
|| "/api/initiate/interest_record".equals(action) String action = sysObject.getString("action");
|| "/api/one_image/share/v3".equals(action) if (null != action) {
|| "/gm_ai/face_app/test_skin".equals(action)) { if ("/api/private_conversation/".equals(action)
jsonObject.put("statistics_action", action); || "/api/initiate/interest_record".equals(action)
return true; || "/api/one_image/share/v3".equals(action)
} || "/gm_ai/face_app/test_skin".equals(action)) {
jsonObject.put("statistics_action", action);
return true;
} }
JSONObject appObject = sysObject.getJSONObject("APP"); }
if (null != appObject) { }
String eventType = appObject.getString("event_type"); JSONObject appObject = jsonObject.getJSONObject("APP");
if (null != eventType) { if (null != appObject) {
if ("validate_order".equals(eventType) String eventType = appObject.getString("event_type");
|| "paid_success".equals(eventType) if (null != eventType) {
|| "add_shopcart".equals(eventType)) { if ("validate_order".equals(eventType)
jsonObject.put("statistics_action", eventType); || "paid_success".equals(eventType)
return true; || "add_shopcart".equals(eventType)) {
} jsonObject.put("statistics_action", eventType);
} return true;
String appAction = appObject.getString("action");
if (null != appAction) {
String[] edits = {"create", "update", "answer"};
if (Arrays.asList(edits).contains(appAction)) {
jsonObject.put("statistics_action", appAction);
return true;
}
String[] interact = {"like", "comment", "collect"};
if (Arrays.asList(interact).contains(appAction)) {
jsonObject.put("statistics_action", appAction);
return true;
}
}
} }
} }
String type = jsonObject.getString("type"); String appAction = appObject.getString("action");
String device = jsonObject.getString("device"); if (null != appAction) {
if (null != type && null != device) { String[] edits = {"create", "update", "answer"};
if ("do_search".equals(type) if (Arrays.asList(edits).contains(appAction)) {
|| "goto_welfare_detail".equals(type) jsonObject.put("statistics_action", appAction);
|| "on_click_card".equals(type) return true;
|| "home_click_section".equals(type)) { }
jsonObject.put("statistics_action", type); String[] interact = {"like", "comment", "collect"};
if (Arrays.asList(interact).contains(appAction)) {
jsonObject.put("statistics_action", appAction);
return true; return true;
} }
} }
} catch (Exception e) {
e.printStackTrace();
return false;
} }
String type = jsonObject.getString("type");
String device = jsonObject.getString("device");
if (null != type && null != device) {
if ("do_search".equals(type)
|| "goto_welfare_detail".equals(type)
|| "on_click_card".equals(type)
|| "home_click_section".equals(type)) {
jsonObject.put("statistics_action", type);
return true;
}
}
} catch (Exception e) {
e.printStackTrace();
return false; return false;
} }
}) return false;
.map(new MapFunction<JSONObject, Tuple2<String, JSONObject>>() { }
@Override });
public Tuple2<String, JSONObject> map(JSONObject jsonObject) throws Exception { //filter01.print();
String statisticsAction = jsonObject.getString("statistics_action"); SingleOutputStreamOperator map02 = filter01.map(new MapFunction<JSONObject, Tuple2<String, JSONObject>>() {
return new Tuple2<String, JSONObject>(statisticsAction, jsonObject) { @Override
}; public Tuple2<String, JSONObject> map(JSONObject jsonObject) throws Exception {
} String statisticsAction = jsonObject.getString("statistics_action");
}) return new Tuple2<String, JSONObject>(statisticsAction, jsonObject) {
.keyBy(0) };
.timeWindow(Time.seconds(windownSize),Time.seconds(slideSize)) }
.process(new ProcessWindowFunction<Tuple2<String,JSONObject>, TblMonitorPortraitShd, Tuple, TimeWindow>() { });
@Override //map02.print();
public void process(Tuple key, Context context, Iterable<Tuple2<String,JSONObject>> elements, Collector<TblMonitorPortraitShd> out) { KeyedStream keyedStream = map02.keyBy(0);
Integer count = 0; //keyedStream.print();
for (Tuple2<String,JSONObject> tuple2 : elements) { WindowedStream windowedStream = keyedStream.timeWindow(Time.seconds(windownSize), Time.seconds(slideSize));
++ count; SingleOutputStreamOperator process = windowedStream.process(new ProcessWindowFunction<Tuple2<String, JSONObject>, TblMonitorPortraitShd, Tuple, TimeWindow>() {
@Override
public void process(Tuple key, Context context, Iterable<Tuple2<String, JSONObject>> elements, Collector<TblMonitorPortraitShd> out) {
Integer count = 0;
Date date = new Date();
long currentTimestamp = DateUtils.getCurrentTimestamp(date);
long tenMinitesAgoTimestamp = DateUtils.getTenMinitesAgoTimestamp(date);
for (Tuple2<String, JSONObject> tuple2 : elements) {
System.err.println(tuple2.f0);
long logTime = 0;
JSONObject jsonObject = tuple2.f1;
String maidianEventTime = jsonObject.getString("create_at");
if (StringUtils.isNotBlank(maidianEventTime)) {
logTime = Long.valueOf(maidianEventTime);
}
String backendEventTime = jsonObject.getString("TIME");
if (StringUtils.isNotBlank(backendEventTime)) {
try {
logTime = dateTimeFormat.parseMillis(backendEventTime);
} catch (IllegalArgumentException e) {
try {
logTime = dateTimeNoMillisFormat.parseMillis(backendEventTime);
} catch (IllegalArgumentException e2) {
e2.printStackTrace();
}
}
}
if (logTime >= tenMinitesAgoTimestamp && logTime <= currentTimestamp) {
++count;
} }
String monitorTime = DateUtils.getCurrentTimeStr();
TblMonitorPortraitShd tblMonitorPortraitShd = new TblMonitorPortraitShd(key.getField(0), count, monitorTime);
out.collect(tblMonitorPortraitShd);
} }
}) String currentTimeStr = DateUtils.getCurrentTimeStr(date);
.addSink(new PortraitShdMysqlSink(outJdbcUrl,maxRetry,retryInteral)) TblMonitorPortraitShd tblMonitorPortraitShd = new TblMonitorPortraitShd(key.getField(0), count, currentTimeStr);
.setParallelism(parallelism); out.collect(tblMonitorPortraitShd);
}
});
process.print();
process.addSink(new PortraitShdMysqlSink(outJdbcUrl,maxRetry,retryInteral))
.setParallelism(parallelism);
} }
} }
...@@ -16,6 +16,8 @@ import org.apache.flink.streaming.api.windowing.time.Time; ...@@ -16,6 +16,8 @@ import org.apache.flink.streaming.api.windowing.time.Time;
import org.apache.flink.streaming.api.windowing.windows.TimeWindow; import org.apache.flink.streaming.api.windowing.windows.TimeWindow;
import org.apache.flink.util.Collector; import org.apache.flink.util.Collector;
import java.util.Date;
/** /**
* @ClassName PortraitMonitorSucOperator * @ClassName PortraitMonitorSucOperator
* @Description: 用户画像成功打点数监控 * @Description: 用户画像成功打点数监控
...@@ -77,11 +79,17 @@ public class PortraitMonitorSucOperator implements BaseOperator{ ...@@ -77,11 +79,17 @@ public class PortraitMonitorSucOperator implements BaseOperator{
@Override @Override
public void process(Tuple key, Context context, Iterable<GmPortraitResult> elements, Collector<TblMonitorPortraitSuc> out) { public void process(Tuple key, Context context, Iterable<GmPortraitResult> elements, Collector<TblMonitorPortraitSuc> out) {
Integer count = 0; Integer count = 0;
Date date = new Date();
long currentTimestamp = DateUtils.getCurrentTimestamp(date);
long tenMinitesAgoTimestamp = DateUtils.getTenMinitesAgoTimestamp(date);
for (GmPortraitResult element : elements) { for (GmPortraitResult element : elements) {
++ count; long logTime = Long.valueOf(element.getLog_time());
if(logTime >= tenMinitesAgoTimestamp && logTime <= currentTimestamp){
++ count;
}
} }
String monitorTime = DateUtils.getCurrentTimeStr(); String currentTimeStr = DateUtils.getCurrentTimeStr(date);
TblMonitorPortraitSuc tblMonitorPortraitSuc = new TblMonitorPortraitSuc(key.getField(0), count, monitorTime); TblMonitorPortraitSuc tblMonitorPortraitSuc = new TblMonitorPortraitSuc(key.getField(0), count, currentTimeStr);
out.collect(tblMonitorPortraitSuc); out.collect(tblMonitorPortraitSuc);
} }
}) })
......
...@@ -2,6 +2,7 @@ package com.gmei.data.monitor.utils; ...@@ -2,6 +2,7 @@ package com.gmei.data.monitor.utils;
import java.text.ParseException; import java.text.ParseException;
import java.text.SimpleDateFormat; import java.text.SimpleDateFormat;
import java.util.Calendar;
import java.util.Date; import java.util.Date;
/** /**
...@@ -17,10 +18,45 @@ public class DateUtils { ...@@ -17,10 +18,45 @@ public class DateUtils {
/** /**
* 获取当前时间字符串 * 获取当前时间字符串
* @param date
* @return * @return
*/ */
public static String getCurrentTimeStr() { public static String getCurrentTimeStr(Date date) {
return new SimpleDateFormat(DATE_FORMATE_YMDHMS).format(new Date()); return new SimpleDateFormat(DATE_FORMATE_YMDHMS).format(date);
}
/**
* 获取十分钟以前的时间字符串
* @param date
* @return
*/
public static String getTenMinitesAgoTimeStr(Date date) {
Calendar calendar = Calendar.getInstance();
calendar.setTime(date);
calendar.add(Calendar.MINUTE,-10);
SimpleDateFormat sdf = new SimpleDateFormat(DATE_FORMATE_YMDHMS);
return sdf.format(calendar.getTime());
}
/**
* 获取当前时间戳
* @param date
* @return
*/
public static long getCurrentTimestamp(Date date) {
return date.getTime();
}
/**
* 获取十分钟以前的时间戳
* @param date
* @return
*/
public static long getTenMinitesAgoTimestamp(Date date) {
Calendar calendar = Calendar.getInstance();
calendar.setTime(date);
calendar.add(Calendar.MINUTE,-10);
return calendar.getTime().getTime();
} }
/** /**
...@@ -33,4 +69,18 @@ public class DateUtils { ...@@ -33,4 +69,18 @@ public class DateUtils {
SimpleDateFormat simpleDateFormat = new SimpleDateFormat(DATE_FORMATE_YMDHMS); SimpleDateFormat simpleDateFormat = new SimpleDateFormat(DATE_FORMATE_YMDHMS);
return simpleDateFormat.parse(dateStr).getTime(); return simpleDateFormat.parse(dateStr).getTime();
} }
/**
* 将秒值转为指定格式的日期
* @param second
* @return
*/
public static String secondToDate(long second) {
Calendar calendar = Calendar.getInstance();
calendar.setTimeInMillis(second * 1000);
Date date = calendar.getTime();
SimpleDateFormat format = new SimpleDateFormat(DATE_FORMATE_YMDHMS);
String dateString = format.format(date);
return dateString;
}
} }
Markdown is supported
0% or
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment