0


Flink 自定义数据源Connector

概述

首先我们先来看一下自定义数据源,Flink系统提供的一些功能

我们可以从下面这个图看出来数据源的source和sink类的集成关系

当我们要实现自定义数据源的时候,我们需要先实现DynamicTableSourceFactory, DynamicTableSinkFactory这两个工厂类,在工厂类里面去实现参数定义和数据源的创建,然后再数据源DynamicTableSource和DynamicTableSink里面去初始化数据源的一些信息,最终在source类型的数据源的ScanRuntimeProvider或者LookupTableSource或者sink类型的数据源的SinkRuntimeProvider实现类里面去实现具体功能。

具体实现

第一步 实现工厂类

Dynamic Table Factories

动态表工厂用于根据目录和会话信息为外部存储系统配置动态表连接器。

  1. org.apache.flink.table.factories.DynamicTableSourceFactory

可以实现构造一个

  1. DynamicTableSource

.

  1. org.apache.flink.table.factories.DynamicTableSinkFactory

可以实现构造一个

  1. DynamicTableSink

.

默认情况下,使用

  1. connector

选项的值作为工厂标识符和 Java 的服务提供者接口来发现工厂。

在 JAR 文件中,可以将对新实现的引用添加到服务文件中:

**需要在resource目录下新建

  1. META-INF/services/目录,并且新建org.apache.flink.table.factories.Factory文件

**

**

  1. META-INF/services/org.apache.flink.table.factories.Factory

**

然后在文件里面添加实现类的路径:com.flink.sql.connector.http.table.HttpDynamicTableFactory,只需要添加这个类路径就行,其他的不需要

如果没有这个文件个文件里面的内容,flink系统识别不到我们自定义的数据源信息

类文件的内容如下:HttpDynamicTableFactory.class

  1. package com.flink.sql.connector.http.table;
  2. import org.apache.flink.api.common.serialization.DeserializationSchema;
  3. import org.apache.flink.api.common.serialization.SerializationSchema;
  4. import org.apache.flink.configuration.ConfigOption;
  5. import org.apache.flink.configuration.ConfigOptions;
  6. import org.apache.flink.configuration.ReadableConfig;
  7. import org.apache.flink.table.api.TableSchema;
  8. import org.apache.flink.table.connector.format.DecodingFormat;
  9. import org.apache.flink.table.connector.format.EncodingFormat;
  10. import org.apache.flink.table.connector.sink.DynamicTableSink;
  11. import org.apache.flink.table.connector.source.DynamicTableSource;
  12. import org.apache.flink.table.data.RowData;
  13. import org.apache.flink.table.factories.DeserializationFormatFactory;
  14. import org.apache.flink.table.factories.DynamicTableSinkFactory;
  15. import org.apache.flink.table.factories.DynamicTableSourceFactory;
  16. import org.apache.flink.table.factories.FactoryUtil;
  17. import org.apache.flink.table.factories.SerializationFormatFactory;
  18. import org.apache.flink.table.types.DataType;
  19. import org.apache.flink.util.Preconditions;
  20. import java.util.HashSet;
  21. import java.util.Optional;
  22. import java.util.Set;
  23. // 我这里是同时实现了source和sink,也可以单独去实现,写在一起可以减少很多重复代码
  24. public class HttpDynamicTableFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory {
  25. /**
  26. 首先定义一些数据源的参数信息,就是连接器的所有参数都需要先定义,这样才能在SQL里面去使用
  27. */
  28. public static final String IDENTIFIER = "http";
  29. public static final ConfigOption<String> URL = ConfigOptions.key("url")
  30. .stringType().noDefaultValue().withDescription("the jdbc database url.");
  31. public static final ConfigOption<String> HEADERS = ConfigOptions.key("headers")
  32. .stringType().noDefaultValue().withDescription("the http header.");
  33. private static final ConfigOption<String> BODY = ConfigOptions.key("body")
  34. .stringType().noDefaultValue().withDescription("the http body params.");
  35. private static final ConfigOption<String> TYPE = ConfigOptions.key("type")
  36. .stringType().noDefaultValue().withDescription("the http type.");
  37. private static final ConfigOption<String> FORMAT = ConfigOptions.key("format")
  38. .stringType().noDefaultValue().withDescription("the http type.");
  39. public HttpDynamicTableFactory() {
  40. }
  41. // 构造source类型的数据源对象
  42. public DynamicTableSource createDynamicTableSource(Context context) {
  43. FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context);
  44. ReadableConfig config = helper.getOptions();
  45. helper.validate();
  46. this.validateConfigOptions(config);
  47. HttpSourceInfo httpSourceInfo = this.getHttpSource(config);
  48. // discover a suitable decoding format
  49. final DecodingFormat<DeserializationSchema<RowData>> decodingFormat = helper.discoverDecodingFormat(
  50. DeserializationFormatFactory.class,
  51. FactoryUtil.FORMAT);
  52. // derive the produced data type (excluding computed columns) from the catalog table
  53. final DataType producedDataType = context.getCatalogTable().getSchema().toPhysicalRowDataType();
  54. return new HttpDynamicTableSource(httpSourceInfo, decodingFormat, producedDataType);
  55. }
  56. // 构造sink类型的数据源对象
  57. @Override
  58. public DynamicTableSink createDynamicTableSink(Context context) {
  59. FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context);
  60. ReadableConfig config = helper.getOptions();
  61. helper.validate();
  62. this.validateConfigOptions(config);
  63. HttpSourceInfo httpSourceInfo = this.getHttpSource(config);
  64. // discover a suitable encoding format
  65. final EncodingFormat<SerializationSchema<RowData>> encodingFormat = helper.discoverEncodingFormat(
  66. SerializationFormatFactory.class,
  67. FactoryUtil.FORMAT);
  68. // derive the produced data type (excluding computed columns) from the catalog table
  69. final DataType producedDataType = context.getCatalogTable().getSchema().toPhysicalRowDataType();
  70. TableSchema tableSchema = context.getCatalogTable().getSchema();
  71. return new HttpDynamicTableSink(httpSourceInfo, encodingFormat, producedDataType, tableSchema);
  72. }
  73. // 获取自定义的HTTP连接器的参数对象,主要用来存HTTP链接的一些参数,后面用来构造HTTP请求使用
  74. private HttpSourceInfo getHttpSource(ReadableConfig readableConfig) {
  75. String url = readableConfig.get(URL);
  76. String headers = readableConfig.get(HEADERS);
  77. String body = readableConfig.get(BODY);
  78. String type = readableConfig.get(TYPE);
  79. return new HttpSourceInfo(url,type, headers, body);
  80. }
  81. // 返回数据源的type字符串,标识这是一个什么数据源
  82. public String factoryIdentifier() {
  83. return IDENTIFIER;
  84. }
  85. // 必填参数需要在这个方法里面去添加
  86. public Set<ConfigOption<?>> requiredOptions() {
  87. Set<ConfigOption<?>> requiredOptions = new HashSet();
  88. requiredOptions.add(URL);
  89. requiredOptions.add(TYPE);
  90. return requiredOptions;
  91. }
  92. // 非必填参数需要在这个方法里面去添加
  93. public Set<ConfigOption<?>> optionalOptions() {
  94. Set<ConfigOption<?>> optionalOptions = new HashSet();
  95. optionalOptions.add(HEADERS);
  96. optionalOptions.add(BODY);
  97. optionalOptions.add(FORMAT);
  98. return optionalOptions;
  99. }
  100. // 参数校验,根据实际情况去实现需要校验哪些参数,比如有些参数有格式校验可以在这里实现,没有可以不实现
  101. private void validateConfigOptions(ReadableConfig config) {
  102. String url = config.get(URL);
  103. Optional<String> urlOp = Optional.of(url);
  104. Preconditions.checkState(urlOp.isPresent(), "Cannot handle such http url: " + url);
  105. String type = config.get(TYPE);
  106. if ("POST".equalsIgnoreCase(type)) {
  107. String body = config.get(BODY);
  108. Optional<String> bodyOp = Optional.of(body);
  109. Preconditions.checkState(bodyOp.isPresent(), "Cannot handle such http post body: " + bodyOp);
  110. }
  111. }
  112. }

第二步:实现数据源类的具体方法

Dynamic Table Source

根据定义,动态表可以随时间变化。

在读取动态表时,内容可以被认为是:

  • 一个更改日志(有限或无限),所有更改都会持续使用,直到更改日志用完。这由ScanTableSource接口表示。
  • 一个不断变化的或非常大的外部表,其内容通常不会被完全读取,而是在必要时查询单个值。这由LookupTableSource 接口表示。

一个类可以同时实现这两个接口。规划器根据指定的查询决定它们的使用。

**扫描表源

  1. ScanTableSource

**

  1. ScanTableSource

在运行时扫描来自外部存储系统的所有行。

扫描的行不必只包含插入,还可以包含更新和删除。因此,表源可用于读取(有限或无限)变更日志。返回的更改日志模式指示计划程序在运行时可以预期的一组更改。

对于常规的批处理场景,源可以发出有限的仅插入行流。

对于常规流式处理方案,源可以发出无限制的仅插入行流。

对于变更数据捕获 (CDC) 方案,源可以发出带有插入、更新和删除行的有界或无界流。

**查找表源

  1. LookupTableSource

**

  1. LookupTableSource

在运行时通过一个或多个键查找外部存储系统的行。

  1. ScanTableSource

相比,源不必读取整个表,并且可以在必要时从(可能不断变化的)外部表中懒惰地获取单个值。

  1. ScanTableSource

相比,

  1. LookupTableSource

当前仅支持发出仅插入更改。

我们的例子是HTTP类型的数据源,所以采用全表扫描获取的类型去实现。

代码实现类:HttpDynamicTableSource.class

  1. package com.flink.sql.connector.http.table;
  2. import com.flink.sql.connector.http.HttpSourceFunction;
  3. import com.flink.sql.connector.http.HttpSourceInfo;
  4. import org.apache.flink.api.common.serialization.DeserializationSchema;
  5. import org.apache.flink.table.connector.ChangelogMode;
  6. import org.apache.flink.table.connector.format.DecodingFormat;
  7. import org.apache.flink.table.connector.source.DynamicTableSource;
  8. import org.apache.flink.table.connector.source.ScanTableSource;
  9. import org.apache.flink.table.connector.source.SourceFunctionProvider;
  10. import org.apache.flink.table.data.RowData;
  11. import org.apache.flink.table.types.DataType;
  12. import java.util.Objects;
  13. public class HttpDynamicTableSource implements ScanTableSource {
  14. private HttpSourceInfo httpSourceInfo;
  15. private DecodingFormat<DeserializationSchema<RowData>> decodingFormat;
  16. private DataType producedDataType;
  17. public HttpDynamicTableSource() {
  18. }
  19. public HttpDynamicTableSource(HttpSourceInfo httpSourceInfo,
  20. DecodingFormat<DeserializationSchema<RowData>> decodingFormat,
  21. DataType producedDataType) {
  22. this.httpSourceInfo = httpSourceInfo;
  23. this.decodingFormat = decodingFormat;
  24. this.producedDataType = producedDataType;
  25. }
  26. @Override
  27. public ChangelogMode getChangelogMode() {
  28. return decodingFormat.getChangelogMode();
  29. }
  30. // 最主要的方法,就是实现这个接口方法,在方法里面返回具体的实现逻辑类的构造
  31. @Override
  32. public ScanRuntimeProvider getScanRuntimeProvider(ScanContext scanContext) {
  33. final DeserializationSchema<RowData> deserializer = decodingFormat.createRuntimeDecoder(
  34. scanContext,
  35. producedDataType);
  36. HttpSourceFunction.Builder builder = HttpSourceFunction.builder()
  37. .setUrl(httpSourceInfo.getUrl()).setBody(httpSourceInfo.getBody())
  38. .setType(httpSourceInfo.getType()).setDeserializer(deserializer);
  39. return SourceFunctionProvider.of(builder.build(), true);
  40. }
  41. @Override
  42. public DynamicTableSource copy() {
  43. return new HttpDynamicTableSource(this.httpSourceInfo, this.decodingFormat, this.producedDataType);
  44. }
  45. @Override
  46. public String asSummaryString() {
  47. return "HTTP Table Source";
  48. }
  49. public boolean equals(Object o) {
  50. if (this == o) {
  51. return true;
  52. } else if (!(o instanceof HttpDynamicTableSource)) {
  53. return false;
  54. } else {
  55. HttpDynamicTableSource that = (HttpDynamicTableSource) o;
  56. return Objects.equals(this.httpSourceInfo, that.httpSourceInfo)
  57. && Objects.equals(this.decodingFormat, that.decodingFormat)
  58. && Objects.equals(this.producedDataType, that.producedDataType);
  59. }
  60. }
  61. public int hashCode() {
  62. return Objects.hash(new Object[]{this.httpSourceInfo, this.decodingFormat, this.producedDataType});
  63. }
  64. }

代码实现类:HttpSourceFunction.class

  1. package com.flink.sql.connector.http;
  2. import com.alibaba.fastjson.JSONObject;
  3. import org.apache.commons.lang3.StringUtils;
  4. import org.apache.flink.api.common.serialization.DeserializationSchema;
  5. import org.apache.flink.api.common.typeinfo.TypeInformation;
  6. import org.apache.flink.api.java.typeutils.ResultTypeQueryable;
  7. import org.apache.flink.configuration.Configuration;
  8. import org.apache.flink.streaming.api.functions.source.RichSourceFunction;
  9. import org.apache.flink.table.data.RowData;
  10. import java.io.IOException;
  11. import java.nio.charset.StandardCharsets;
  12. public class HttpSourceFunction extends RichSourceFunction<RowData> implements ResultTypeQueryable<RowData> {
  13. private String url;
  14. private String body;
  15. private String type;
  16. private String result;
  17. private static final String HTTP_POST = "POST";
  18. private static final String HTTP_GET = "GET";
  19. private transient boolean hasNext;
  20. private DeserializationSchema<RowData> deserializer;
  21. public HttpSourceFunction() {
  22. }
  23. public HttpSourceFunction(String url, String body, String type, DeserializationSchema<RowData> deserializer) {
  24. this.url = url;
  25. this.body = body;
  26. this.type = type;
  27. this.deserializer = deserializer;
  28. }
  29. @Override
  30. public void open(Configuration parameters) throws Exception {
  31. deserializer.open(() -> getRuntimeContext().getMetricGroup());
  32. }
  33. @Override
  34. public void close() throws IOException {
  35. }
  36. @Override
  37. public TypeInformation<RowData> getProducedType() {
  38. return deserializer.getProducedType();
  39. }
  40. public static HttpSourceFunction.Builder builder() {
  41. return new HttpSourceFunction.Builder();
  42. }
  43. // 重点关注地方,在run方法里面去实现具体调用HTTP请求获取数据的逻辑。我这里是实现了一个工具类去调用HTTP的各种POST或者GET请求,大家可以自己去实现,功能就是执行请求获取数据,具体代码就不贴了。
  44. @Override
  45. public void run(SourceContext<RowData> sourceContext) throws Exception {
  46. // open and consume from socket
  47. try {
  48. String response = "{}";
  49. if (StringUtils.isNotBlank(type) && HTTP_GET.equals(type.toUpperCase())) {
  50. response = DtHttpClient.get(url);
  51. } else {
  52. response = DtHttpClient.post(url, body);
  53. }
  54. JSONObject jsonObject = JSONObject.parseObject(response);
  55. this.result = jsonObject.getString("response");
  56. sourceContext.collect(deserializer.deserialize(result.getBytes(StandardCharsets.UTF_8)));
  57. } catch (Throwable t) {
  58. t.printStackTrace(); // print and continue
  59. }
  60. }
  61. @Override
  62. public void cancel() {
  63. }
  64. public static class Builder {
  65. private String url;
  66. private String body;
  67. private String type;
  68. private DeserializationSchema<RowData> deserializer;
  69. public Builder () {
  70. }
  71. public Builder setUrl(String url) {
  72. this.url = url;
  73. return this;
  74. }
  75. public Builder setBody(String body) {
  76. this.body = body;
  77. return this;
  78. }
  79. public Builder setType(String type) {
  80. this.type = type;
  81. return this;
  82. }
  83. public Builder setDeserializer(DeserializationSchema<RowData> deserializer) {
  84. this.deserializer = deserializer;
  85. return this;
  86. }
  87. public HttpSourceFunction build() {
  88. if (StringUtils.isBlank(url) || StringUtils.isBlank(body) || StringUtils.isBlank(type)) {
  89. throw new IllegalArgumentException("params has null");
  90. }
  91. return new HttpSourceFunction(this.url, this.body, this.type, this.deserializer);
  92. }
  93. }
  94. }

Dynamic Table Sink

根据定义,动态表可以随时间变化。

在编写动态表时,可以始终将内容视为更改日志(有限或无限),其中所有更改都被连续写出,直到更改日志用完为止。返回的更改日志模式 指示接收器在运行时接受的更改集。

对于常规批处理场景,接收器可以仅接受仅插入行并写出有界流。

对于常规的流式处理方案,接收器只能接受仅插入行,并且可以写出无界流。

对于变更数据捕获 (CDC) 场景,接收器可以使用插入、更新和删除行写出有界或无界流。

代码实现类:HttpDynamicTableSink.class

  1. package com.flink.sql.connector.http.table;
  2. import com.flink.sql.connector.http.HttpSinkFunction;
  3. import com.flink.sql.connector.http.HttpSourceInfo;
  4. import org.apache.flink.api.common.serialization.SerializationSchema;
  5. import org.apache.flink.table.api.TableSchema;
  6. import org.apache.flink.table.connector.ChangelogMode;
  7. import org.apache.flink.table.connector.format.EncodingFormat;
  8. import org.apache.flink.table.connector.sink.DynamicTableSink;
  9. import org.apache.flink.table.connector.sink.SinkFunctionProvider;
  10. import org.apache.flink.table.data.RowData;
  11. import org.apache.flink.table.types.DataType;
  12. import java.util.Objects;
  13. public class HttpDynamicTableSink implements DynamicTableSink {
  14. private HttpSourceInfo httpSourceInfo;
  15. private EncodingFormat<SerializationSchema<RowData>> encodingFormat;
  16. private DataType producedDataType;
  17. private TableSchema tableSchema;
  18. public HttpDynamicTableSink() {
  19. }
  20. public HttpDynamicTableSink(HttpSourceInfo httpSourceInfo,
  21. EncodingFormat<SerializationSchema<RowData>> encodingFormat,
  22. DataType producedDataType,
  23. TableSchema tableSchema) {
  24. this.httpSourceInfo = httpSourceInfo;
  25. this.encodingFormat = encodingFormat;
  26. this.producedDataType = producedDataType;
  27. this.tableSchema = tableSchema;
  28. }
  29. @Override
  30. public ChangelogMode getChangelogMode(ChangelogMode changelogMode) {
  31. return changelogMode;
  32. }
  33. // 重点关注地方,主要通过这个方法去构造输出数据源的实现类
  34. @Override
  35. public SinkRuntimeProvider getSinkRuntimeProvider(Context context) {
  36. final SerializationSchema<RowData> deserializer = encodingFormat.createRuntimeEncoder(
  37. context,
  38. producedDataType);
  39. HttpSinkFunction httpSinkFunction =
  40. HttpSinkFunction.builder().setUrl(httpSourceInfo.getUrl())
  41. .setBody(httpSourceInfo.getBody()).setDeserializer(deserializer)
  42. .setType(httpSourceInfo.getType()).setFields(tableSchema.getFieldNames())
  43. .setDataTypes(tableSchema.getFieldDataTypes()).build();
  44. return SinkFunctionProvider.of(httpSinkFunction);
  45. }
  46. @Override
  47. public DynamicTableSink copy() {
  48. return new HttpDynamicTableSink(this.httpSourceInfo, this.encodingFormat,
  49. this.producedDataType, this.tableSchema);
  50. }
  51. @Override
  52. public String asSummaryString() {
  53. return "HTTP Table Sink";
  54. }
  55. public boolean equals(Object o) {
  56. if (this == o) {
  57. return true;
  58. } else if (!(o instanceof HttpDynamicTableSink)) {
  59. return false;
  60. } else {
  61. HttpDynamicTableSink that = (HttpDynamicTableSink) o;
  62. return Objects.equals(this.httpSourceInfo, that.httpSourceInfo)
  63. && Objects.equals(this.encodingFormat, that.encodingFormat)
  64. && Objects.equals(this.producedDataType, that.producedDataType);
  65. }
  66. }
  67. public int hashCode() {
  68. return Objects.hash(new Object[]{this.httpSourceInfo, this.encodingFormat,
  69. this.producedDataType, this.tableSchema});
  70. }
  71. }

代码实现类:HttpSinkFunction.class

  1. package com.flink.sql.connector.http;
  2. import com.flink.sql.util.PluginUtil;
  3. import org.apache.commons.lang3.StringUtils;
  4. import org.apache.flink.api.common.serialization.SerializationSchema;
  5. import org.apache.flink.configuration.Configuration;
  6. import org.apache.flink.streaming.api.functions.sink.RichSinkFunction;
  7. import org.apache.flink.table.data.RowData;
  8. import org.apache.flink.table.types.DataType;
  9. import org.apache.flink.table.types.logical.DecimalType;
  10. import org.apache.flink.table.types.logical.LogicalType;
  11. import org.apache.flink.table.types.logical.TimestampType;
  12. import java.io.IOException;
  13. import java.sql.Date;
  14. import java.text.SimpleDateFormat;
  15. import java.time.LocalDate;
  16. import java.util.HashMap;
  17. public class HttpSinkFunction extends RichSinkFunction<RowData> {
  18. private String url;
  19. private String body;
  20. private String type;
  21. private SerializationSchema<RowData> serializer;
  22. private String[] fields;
  23. private DataType[] dataTypes;
  24. private final SimpleDateFormat dateFormatter = new SimpleDateFormat("yyyy-MM-dd");
  25. private final SimpleDateFormat dateTimeFormatter = new SimpleDateFormat("yyyy-MM-dd HH:mm:ss");
  26. private static final long serialVersionUID = 1L;
  27. public HttpSinkFunction() {
  28. }
  29. public HttpSinkFunction(String url, String body, String type,
  30. SerializationSchema<RowData> serializer,
  31. String[] fields,
  32. DataType[] dataTypes) {
  33. this.url = url;
  34. this.body = body;
  35. this.type = type;
  36. this.serializer = serializer;
  37. this.fields = fields;
  38. this.dataTypes = dataTypes;
  39. }
  40. // 重点关注,这个invoke方法实现对数据的写出,参数RowData value就是需要输出的数据,这个对象里面具体有多少数据是不确定的,因为默认是流式输出,如果需要考虑性能问题(并且对于实时性没有太高要求),可以自定义实现批量输出,先把这个里面的数据缓存起来,然后当一定时间,或者数据量达到一定阈值的时候再去调研接口输出数据。
  41. @Override
  42. public void invoke(RowData value, Context context) throws Exception {
  43. Object[] objValue = transform(value);
  44. HashMap<String, Object> map = new HashMap<>();
  45. for (int i = 0; i < fields.length; i++) {
  46. map.put(fields[i], objValue[i]);
  47. }
  48. String body = PluginUtil.objectToString(map);
  49. DtHttpClient.post(url, body);
  50. }
  51. @Override
  52. public void open(Configuration parameters) throws Exception {
  53. serializer.open(() -> getRuntimeContext().getMetricGroup());
  54. }
  55. @Override
  56. public void close() throws IOException {
  57. }
  58. public static HttpSinkFunction.Builder builder() {
  59. return new HttpSinkFunction.Builder();
  60. }
  61. public static class Builder {
  62. private String url;
  63. private String body;
  64. private String type;
  65. private SerializationSchema<RowData> serializer;
  66. private String[] fields;
  67. private DataType[] dataTypes;
  68. public Builder () {
  69. }
  70. public Builder setUrl(String url) {
  71. this.url = url;
  72. return this;
  73. }
  74. public Builder setFields(String[] fields) {
  75. this.fields = fields;
  76. return this;
  77. }
  78. public Builder setBody(String body) {
  79. this.body = body;
  80. return this;
  81. }
  82. public Builder setType(String type) {
  83. this.type = type;
  84. return this;
  85. }
  86. public Builder setDataTypes(DataType[] dataTypes) {
  87. this.dataTypes = dataTypes;
  88. return this;
  89. }
  90. public Builder setDeserializer(SerializationSchema<RowData> serializer) {
  91. this.serializer = serializer;
  92. return this;
  93. }
  94. public HttpSinkFunction build() {
  95. if (StringUtils.isBlank(url) || StringUtils.isBlank(body) || StringUtils.isBlank(type)) {
  96. throw new IllegalArgumentException("params has null");
  97. }
  98. return new HttpSinkFunction(this.url, this.body, this.type, this.serializer,
  99. this.fields, this.dataTypes);
  100. }
  101. }
  102. // 这个方法是用来把RowData对象转换为HTTP接口能够识别的JSON对象的,因为默认HTTP接口不能识别这种复杂对象并且转换为我们常用的JSON对象,所以需要我们自己去解析。当然直接把这个对象丢给HTTP也是可以的,那么就需要在接收方去解析RowData对象,但是默认来说,肯定解析为更通用的类型最合适
  103. public Object[] transform(RowData record) {
  104. Object[] values = new Object[dataTypes.length];
  105. int idx = 0;
  106. int var6 = dataTypes.length;
  107. for (int i = 0; i < var6; ++i) {
  108. DataType dataType = dataTypes[i];
  109. values[idx] = this.typeConvertion(dataType.getLogicalType(), record, idx);
  110. ++idx;
  111. }
  112. return values;
  113. }
  114. private Object typeConvertion(LogicalType type, RowData record, int pos) {
  115. if (record.isNullAt(pos)) {
  116. return null;
  117. } else {
  118. switch (type.getTypeRoot()) {
  119. case BOOLEAN:
  120. return record.getBoolean(pos) ? 1L : 0L;
  121. case TINYINT:
  122. return record.getByte(pos);
  123. case SMALLINT:
  124. return record.getShort(pos);
  125. case INTEGER:
  126. return record.getInt(pos);
  127. case BIGINT:
  128. return record.getLong(pos);
  129. case FLOAT:
  130. return record.getFloat(pos);
  131. case DOUBLE:
  132. return record.getDouble(pos);
  133. case CHAR:
  134. case VARCHAR:
  135. return record.getString(pos).toString();
  136. case DATE:
  137. return this.dateFormatter.format(Date.valueOf(LocalDate.ofEpochDay(record.getInt(pos))));
  138. case TIMESTAMP_WITHOUT_TIME_ZONE:
  139. int timestampPrecision = ((TimestampType) type).getPrecision();
  140. return this.dateTimeFormatter.format(new Date(record.getTimestamp(pos, timestampPrecision)
  141. .toTimestamp().getTime()));
  142. case DECIMAL:
  143. int decimalPrecision = ((DecimalType) type).getPrecision();
  144. int decimalScale = ((DecimalType) type).getScale();
  145. return record.getDecimal(pos, decimalPrecision, decimalScale).toBigDecimal();
  146. default:
  147. throw new UnsupportedOperationException("Unsupported type:" + type);
  148. }
  149. }
  150. }
  151. }

最后在FlinkSQL里面使用:

  1. CREATE TABLE source_table(
  2. id BIGINT,
  3. name STRING,
  4. descs STRING,
  5. valuess DOUBLE
  6. )
  7. WITH(
  8. 'connector' ='http',
  9. 'url' ='http//10.36.248.26:8080/test_flink?username=test',
  10. 'type' ='GET'
  11. );
  12. CREATE TABLE sink_table(
  13. id BIGINT,
  14. name STRING,
  15. descs STRING,
  16. valuess DOUBLE )
  17. WITH(
  18. 'connector' ='http',
  19. 'url' ='http//10.36.248.26:8080/post_flink',
  20. 'type' ='POST');

源码地址:https://download.csdn.net/download/tianhouquan/87424283


本文转载自: https://blog.csdn.net/tianhouquan/article/details/124752408
版权归原作者 Yaphets丶混世大魔王 所有, 如有侵权,请联系我们删除。

“Flink 自定义数据源Connector”的评论:

还没有评论