@PublicEvolving public final class FactoryUtil extends Object
Factory
s.Modifier and Type | Class and Description |
---|---|
static class |
FactoryUtil.CatalogFactoryHelper
Helper utility for validating all options for a
CatalogFactory . |
static class |
FactoryUtil.CatalogStoreFactoryHelper
Helper utility for validating all options for a
CatalogStoreFactory . |
static class |
FactoryUtil.DefaultCatalogContext
Default implementation of
CatalogFactory.Context . |
static class |
FactoryUtil.DefaultCatalogStoreContext
Default implementation of
CatalogStoreFactory.Context . |
static class |
FactoryUtil.DefaultDynamicTableContext
Default implementation of
DynamicTableFactory.Context . |
static class |
FactoryUtil.DefaultModuleContext
Default implementation of
ModuleFactory.Context . |
static class |
FactoryUtil.FactoryHelper<F extends Factory>
Base helper utility for validating all options for a
Factory . |
static class |
FactoryUtil.ModuleFactoryHelper
Helper utility for validating all options for a
ModuleFactory . |
static class |
FactoryUtil.TableFactoryHelper
Helper utility for discovering formats and validating all options for a
DynamicTableFactory . |
Modifier and Type | Field and Description |
---|---|
static ConfigOption<String> |
CONNECTOR |
static ConfigOption<String> |
FORMAT |
static String |
FORMAT_SUFFIX
Suffix for keys of
ConfigOption in case a connector requires multiple formats (e.g. |
static String |
PLACEHOLDER_SYMBOL
The placeholder symbol to be used for keys of options which can be templated.
|
static ConfigOption<Integer> |
PROPERTY_VERSION
Describes the property version.
|
static ConfigOption<Integer> |
SINK_PARALLELISM |
static ConfigOption<Duration> |
SOURCE_IDLE_TIMEOUT |
static ConfigOption<Integer> |
SOURCE_PARALLELISM |
static ConfigOption<List<String>> |
SQL_GATEWAY_ENDPOINT_TYPE |
static ConfigOption<String> |
WATERMARK_ALIGNMENT_GROUP |
static ConfigOption<Duration> |
WATERMARK_ALIGNMENT_MAX_DRIFT |
static ConfigOption<Duration> |
WATERMARK_ALIGNMENT_UPDATE_INTERVAL |
static ConfigOption<WatermarkEmitStrategy> |
WATERMARK_EMIT_STRATEGY |
Modifier and Type | Method and Description |
---|---|
static Optional<String> |
checkWatermarkOptions(ReadableConfig conf)
Check watermark-related options and return error messages.
|
static Catalog |
createCatalog(String catalogName,
Map<String,String> options,
ReadableConfig configuration,
ClassLoader classLoader)
Attempts to discover an appropriate catalog factory and creates an instance of the catalog.
|
static FactoryUtil.CatalogFactoryHelper |
createCatalogFactoryHelper(CatalogFactory factory,
CatalogFactory.Context context)
Creates a utility that helps validating options for a
CatalogFactory . |
static FactoryUtil.CatalogStoreFactoryHelper |
createCatalogStoreFactoryHelper(CatalogStoreFactory factory,
CatalogStoreFactory.Context context)
Creates a utility that helps validating options for a
CatalogStoreFactory . |
static DynamicTableSink |
createDynamicTableSink(DynamicTableSinkFactory preferredFactory,
ObjectIdentifier objectIdentifier,
ResolvedCatalogTable catalogTable,
Map<String,String> enrichmentOptions,
ReadableConfig configuration,
ClassLoader classLoader,
boolean isTemporary)
Creates a
DynamicTableSink from a CatalogTable . |
static DynamicTableSink |
createDynamicTableSink(DynamicTableSinkFactory preferredFactory,
ObjectIdentifier objectIdentifier,
ResolvedCatalogTable catalogTable,
ReadableConfig configuration,
ClassLoader classLoader,
boolean isTemporary)
|
static DynamicTableSource |
createDynamicTableSource(DynamicTableSourceFactory preferredFactory,
ObjectIdentifier objectIdentifier,
ResolvedCatalogTable catalogTable,
Map<String,String> enrichmentOptions,
ReadableConfig configuration,
ClassLoader classLoader,
boolean isTemporary)
Creates a
DynamicTableSource from a CatalogTable . |
static DynamicTableSource |
createDynamicTableSource(DynamicTableSourceFactory preferredFactory,
ObjectIdentifier objectIdentifier,
ResolvedCatalogTable catalogTable,
ReadableConfig configuration,
ClassLoader classLoader,
boolean isTemporary)
|
static Module |
createModule(String moduleName,
Map<String,String> options,
ReadableConfig configuration,
ClassLoader classLoader)
Discovers a matching module factory and creates an instance of it.
|
static FactoryUtil.ModuleFactoryHelper |
createModuleFactoryHelper(ModuleFactory factory,
ModuleFactory.Context context)
Creates a utility that helps validating options for a
ModuleFactory . |
static FactoryUtil.TableFactoryHelper |
createTableFactoryHelper(DynamicTableFactory factory,
DynamicTableFactory.Context context)
Creates a utility that helps in discovering formats, merging options with
DynamicTableFactory.Context.getEnrichmentOptions() and validating them all for a DynamicTableFactory . |
static DynamicTableSink |
createTableSink(Catalog catalog,
ObjectIdentifier objectIdentifier,
ResolvedCatalogTable catalogTable,
ReadableConfig configuration,
ClassLoader classLoader,
boolean isTemporary)
|
static DynamicTableSource |
createTableSource(Catalog catalog,
ObjectIdentifier objectIdentifier,
ResolvedCatalogTable catalogTable,
ReadableConfig configuration,
ClassLoader classLoader,
boolean isTemporary)
|
static <T extends Factory> |
discoverFactory(ClassLoader classLoader,
Class<T> factoryClass,
String factoryIdentifier)
Discovers a factory using the given factory base class and identifier.
|
static <T extends DynamicTableFactory> |
getDynamicTableFactory(Class<T> factoryClass,
Catalog catalog)
Returns the
DynamicTableFactory via Catalog . |
static String |
getFormatPrefix(ConfigOption<String> formatOption,
String formatIdentifier)
Returns the required option prefix for options of the given format.
|
static void |
validateFactoryOptions(Factory factory,
ReadableConfig options)
Validates the required and optional
ConfigOption s of a factory. |
static void |
validateFactoryOptions(Set<ConfigOption<?>> requiredOptions,
Set<ConfigOption<?>> optionalOptions,
ReadableConfig options)
Validates the required options and optional options.
|
static void |
validateUnconsumedKeys(String factoryIdentifier,
Set<String> allOptionKeys,
Set<String> consumedOptionKeys)
Validates unconsumed option keys.
|
static void |
validateUnconsumedKeys(String factoryIdentifier,
Set<String> allOptionKeys,
Set<String> consumedOptionKeys,
Set<String> deprecatedOptionKeys)
Validates unconsumed option keys.
|
static void |
validateWatermarkOptions(String factoryIdentifier,
ReadableConfig conf)
Validate watermark options from table options.
|
public static final ConfigOption<Integer> PROPERTY_VERSION
public static final ConfigOption<String> CONNECTOR
public static final ConfigOption<String> FORMAT
public static final ConfigOption<Integer> SINK_PARALLELISM
public static final ConfigOption<List<String>> SQL_GATEWAY_ENDPOINT_TYPE
public static final ConfigOption<Integer> SOURCE_PARALLELISM
public static final ConfigOption<WatermarkEmitStrategy> WATERMARK_EMIT_STRATEGY
public static final ConfigOption<String> WATERMARK_ALIGNMENT_GROUP
public static final ConfigOption<Duration> WATERMARK_ALIGNMENT_MAX_DRIFT
public static final ConfigOption<Duration> WATERMARK_ALIGNMENT_UPDATE_INTERVAL
public static final ConfigOption<Duration> SOURCE_IDLE_TIMEOUT
public static final String FORMAT_SUFFIX
ConfigOption
in case a connector requires multiple formats (e.g.
for both key and value).
See createTableFactoryHelper(DynamicTableFactory, DynamicTableFactory.Context)
for more information.
public static final String PLACEHOLDER_SYMBOL
Factory
for details.public static DynamicTableSource createDynamicTableSource(@Nullable DynamicTableSourceFactory preferredFactory, ObjectIdentifier objectIdentifier, ResolvedCatalogTable catalogTable, Map<String,String> enrichmentOptions, ReadableConfig configuration, ClassLoader classLoader, boolean isTemporary)
DynamicTableSource
from a CatalogTable
.
If is passed, the table source is created from that factory.
Otherwise, an attempt is made to discover a matching factory using Java SPI (see Factory
for details).
@Deprecated public static DynamicTableSource createDynamicTableSource(@Nullable DynamicTableSourceFactory preferredFactory, ObjectIdentifier objectIdentifier, ResolvedCatalogTable catalogTable, ReadableConfig configuration, ClassLoader classLoader, boolean isTemporary)
@Deprecated public static DynamicTableSource createTableSource(@Nullable Catalog catalog, ObjectIdentifier objectIdentifier, ResolvedCatalogTable catalogTable, ReadableConfig configuration, ClassLoader classLoader, boolean isTemporary)
public static DynamicTableSink createDynamicTableSink(@Nullable DynamicTableSinkFactory preferredFactory, ObjectIdentifier objectIdentifier, ResolvedCatalogTable catalogTable, Map<String,String> enrichmentOptions, ReadableConfig configuration, ClassLoader classLoader, boolean isTemporary)
DynamicTableSink
from a CatalogTable
.
If is passed, the table sink is created from that factory.
Otherwise, an attempt is made to discover a matching factory using Java SPI (see Factory
for details).
@Deprecated public static DynamicTableSink createDynamicTableSink(@Nullable DynamicTableSinkFactory preferredFactory, ObjectIdentifier objectIdentifier, ResolvedCatalogTable catalogTable, ReadableConfig configuration, ClassLoader classLoader, boolean isTemporary)
@Deprecated public static DynamicTableSink createTableSink(@Nullable Catalog catalog, ObjectIdentifier objectIdentifier, ResolvedCatalogTable catalogTable, ReadableConfig configuration, ClassLoader classLoader, boolean isTemporary)
public static FactoryUtil.CatalogFactoryHelper createCatalogFactoryHelper(CatalogFactory factory, CatalogFactory.Context context)
CatalogFactory
.
Note: This utility checks for left-over options in the final step.
public static FactoryUtil.CatalogStoreFactoryHelper createCatalogStoreFactoryHelper(CatalogStoreFactory factory, CatalogStoreFactory.Context context)
CatalogStoreFactory
.
Note: This utility checks for left-over options in the final step.
public static FactoryUtil.ModuleFactoryHelper createModuleFactoryHelper(ModuleFactory factory, ModuleFactory.Context context)
ModuleFactory
.
Note: This utility checks for left-over options in the final step.
public static FactoryUtil.TableFactoryHelper createTableFactoryHelper(DynamicTableFactory factory, DynamicTableFactory.Context context)
DynamicTableFactory.Context.getEnrichmentOptions()
and validating them all for a DynamicTableFactory
.
The following example sketches the usage:
// in createDynamicTableSource()
helper = FactoryUtil.createTableFactoryHelper(this, context);
keyFormat = helper.discoverDecodingFormat(DeserializationFormatFactory.class, KEY_FORMAT);
valueFormat = helper.discoverDecodingFormat(DeserializationFormatFactory.class, VALUE_FORMAT);
helper.validate();
... // construct connector with discovered formats
Note: The format option parameter of FactoryUtil.TableFactoryHelper.discoverEncodingFormat(Class, ConfigOption)
and FactoryUtil.TableFactoryHelper.discoverDecodingFormat(Class, ConfigOption)
must be FORMAT
or
end with FORMAT_SUFFIX
. The discovery logic will replace 'format' with the factory
identifier value as the format prefix. For example, assuming the identifier is 'json', if the
format option key is 'format', then the format prefix is 'json.'. If the format option key is
'value.format', then the format prefix is 'value.json'. The format prefix is used to project
the options for the format factory.
Note: When created, this utility merges the options from DynamicTableFactory.Context.getEnrichmentOptions()
using DynamicTableFactory.forwardOptions()
. When invoking FactoryUtil.FactoryHelper.validate()
,
this utility checks for left-over options in the final step.
public static Catalog createCatalog(String catalogName, Map<String,String> options, ReadableConfig configuration, ClassLoader classLoader)
This first uses the legacy TableFactory
stack to discover a matching CatalogFactory
. If none is found, it falls back to the new stack using Factory
instead.
public static Module createModule(String moduleName, Map<String,String> options, ReadableConfig configuration, ClassLoader classLoader)
This first uses the legacy TableFactory
stack to discover a matching ModuleFactory
. If none is found, it falls back to the new stack using Factory
instead.
public static <T extends Factory> T discoverFactory(ClassLoader classLoader, Class<T> factoryClass, String factoryIdentifier)
This method is meant for cases where createTableFactoryHelper(DynamicTableFactory,
DynamicTableFactory.Context)
createTableSource(Catalog, ObjectIdentifier,
ResolvedCatalogTable, ReadableConfig, ClassLoader, boolean)
, and createTableSink(Catalog, ObjectIdentifier, ResolvedCatalogTable, ReadableConfig,
ClassLoader, boolean)
are not applicable.
public static void validateFactoryOptions(Factory factory, ReadableConfig options)
ConfigOption
s of a factory.
Note: It does not check for left-over options.
public static void validateFactoryOptions(Set<ConfigOption<?>> requiredOptions, Set<ConfigOption<?>> optionalOptions, ReadableConfig options)
Note: It does not check for left-over options.
public static void validateUnconsumedKeys(String factoryIdentifier, Set<String> allOptionKeys, Set<String> consumedOptionKeys, Set<String> deprecatedOptionKeys)
public static void validateUnconsumedKeys(String factoryIdentifier, Set<String> allOptionKeys, Set<String> consumedOptionKeys)
public static String getFormatPrefix(ConfigOption<String> formatOption, String formatIdentifier)
public static <T extends DynamicTableFactory> Optional<T> getDynamicTableFactory(Class<T> factoryClass, @Nullable Catalog catalog)
DynamicTableFactory
via Catalog
.public static void validateWatermarkOptions(String factoryIdentifier, ReadableConfig conf)
factoryIdentifier
- identifier of tableconf
- table optionspublic static Optional<String> checkWatermarkOptions(ReadableConfig conf)
conf
- table optionsCopyright © 2014–2024 The Apache Software Foundation. All rights reserved.