/*
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.flink.api.java.typeutils;
import org.apache.flink.annotation.Internal;
import org.apache.flink.annotation.Public;
import org.apache.flink.annotation.PublicEvolving;
import org.apache.flink.api.common.functions.AggregateFunction;
import org.apache.flink.api.common.functions.CoGroupFunction;
import org.apache.flink.api.common.functions.CrossFunction;
import org.apache.flink.api.common.functions.FlatJoinFunction;
import org.apache.flink.api.common.functions.FlatMapFunction;
import org.apache.flink.api.common.functions.Function;
import org.apache.flink.api.common.functions.GroupCombineFunction;
import org.apache.flink.api.common.functions.GroupReduceFunction;
import org.apache.flink.api.common.functions.InvalidTypesException;
import org.apache.flink.api.common.functions.JoinFunction;
import org.apache.flink.api.common.functions.MapFunction;
import org.apache.flink.api.common.functions.MapPartitionFunction;
import org.apache.flink.api.common.functions.Partitioner;
import org.apache.flink.api.common.io.InputFormat;
import org.apache.flink.api.common.typeinfo.BasicArrayTypeInfo;
import org.apache.flink.api.common.typeinfo.BasicTypeInfo;
import org.apache.flink.api.common.typeinfo.PrimitiveArrayTypeInfo;
import org.apache.flink.api.common.typeinfo.SqlTimeTypeInfo;
import org.apache.flink.api.common.typeinfo.TypeInfo;
import org.apache.flink.api.common.typeinfo.TypeInfoFactory;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.common.typeutils.CompositeType;
import org.apache.flink.api.java.functions.KeySelector;
import org.apache.flink.api.java.tuple.Tuple;
import org.apache.flink.api.java.tuple.Tuple0;
import org.apache.flink.api.java.typeutils.TypeExtractionUtils.LambdaExecutable;
import org.apache.flink.types.Row;
import org.apache.flink.types.Value;
import org.apache.flink.util.InstantiationUtil;
import org.apache.flink.util.Preconditions;
import org.apache.commons.lang3.ClassUtils;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.lang.reflect.Constructor;
import java.lang.reflect.Field;
import java.lang.reflect.GenericArrayType;
import java.lang.reflect.InvocationTargetException;
import java.lang.reflect.Method;
import java.lang.reflect.Modifier;
import java.lang.reflect.ParameterizedType;
import java.lang.reflect.Type;
import java.lang.reflect.TypeVariable;
import java.util.ArrayList;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.checkAndExtractLambda;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.getAllDeclaredMethods;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.getTypeHierarchy;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.hasSuperclass;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.isClassType;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.sameTypeVars;
import static org.apache.flink.api.java.typeutils.TypeExtractionUtils.typeToClass;
import static org.apache.flink.util.Preconditions.checkNotNull;
/**
* A utility for reflection analysis on classes, to determine the return type of implementations of
* transformation functions.
*
* <p>NOTES FOR USERS OF THIS CLASS: Automatic type extraction is a hacky business that depends on a
* lot of variables such as generics, compiler, interfaces, etc. The type extraction fails regularly
* with either {@link MissingTypeInfo} or hard exceptions. Whenever you use methods of this class,
* make sure to provide a way to pass custom type information as a fallback.
*/
@Public
public class TypeExtractor {
/*
* NOTE: Most methods of the TypeExtractor work with a so-called "typeHierarchy".
* The type hierarchy describes all types (Classes, ParameterizedTypes, TypeVariables etc. ) and intermediate
* types from a given type of a function or type (e.g. MyMapper, Tuple2) until a current type
* (depends on the method, e.g. MyPojoFieldType).
*
* Thus, it fully qualifies types until tuple/POJO field level.
*
* A typical typeHierarchy could look like:
*
* UDF: MyMapFunction.class
* top-level UDF: MyMapFunctionBase.class
* RichMapFunction: RichMapFunction.class
* MapFunction: MapFunction.class
* Function's OUT: Tuple1<MyPojo>
* user-defined POJO: MyPojo.class
* user-defined top-level POJO: MyPojoBase.class
* POJO field: Tuple1<String>
* Field type: String.class
*
*/
/** The name of the class representing Hadoop's writable */
private static final String HADOOP_WRITABLE_CLASS = "org.apache.hadoop.io.Writable";
private static final String HADOOP_WRITABLE_TYPEINFO_CLASS =
"org.apache.flink.api.java.typeutils.WritableTypeInfo";
private static final String AVRO_SPECIFIC_RECORD_BASE_CLASS =
"org.apache.avro.specific.SpecificRecordBase";
private static final Logger LOG = LoggerFactory.getLogger(TypeExtractor.class);
private static final String GENERIC_TYPE_DOC_HINT =
"Please read the Flink documentation on \"Data Types & Serialization\" for details of the effect on performance and schema evolution.";
public static final int[] NO_INDEX = new int[] {};
protected TypeExtractor() {
// only create instances for special use cases
}
// --------------------------------------------------------------------------------------------
// Function specific methods
// --------------------------------------------------------------------------------------------
@PublicEvolving
public static <IN, OUT> TypeInformation<OUT> getMapReturnTypes(
MapFunction<IN, OUT> mapInterface, TypeInformation<IN> inType) {
return getMapReturnTypes(mapInterface, inType, null, false);
}
@PublicEvolving
public static <IN, OUT> TypeInformation<OUT> getMapReturnTypes(
MapFunction<IN, OUT> mapInterface,
TypeInformation<IN> inType,
String functionName,
boolean allowMissing) {
return getUnaryOperatorReturnType(
mapInterface,
MapFunction.class,
0,
1,
NO_INDEX,
inType,
functionName,
allowMissing);
}
@PublicEvolving
public static <IN, OUT> TypeInformation<OUT> getFlatMapReturnTypes(
FlatMapFunction<IN, OUT> flatMapInterface, TypeInformation<IN> inType) {
return getFlatMapReturnTypes(flatMapInterface, inType, null, false);
}
@PublicEvolving
public static <IN, OUT> TypeInformation<OUT> getFlatMapReturnTypes(
FlatMapFunction<IN, OUT> flatMapInterface,
TypeInformation<IN> inType,
String functionName,
boolean allowMissing) {
return getUnaryOperatorReturnType(
flatMapInterface,
FlatMapFunction.class,
0,
1,
new int[] {1, 0},
in
没有合适的资源?快使用搜索试试~ 我知道了~
flink1.18.0 源码用于666
共2000个文件
java:1877个
xml:39个
sh:31个
需积分: 0 0 下载量 116 浏览量
2024-12-04
16:39:50
上传
评论
收藏 55.55MB ZIP 举报
温馨提示
flink1.18
资源推荐
资源详情
资源评论
收起资源包目录
flink1.18.0 源码用于666 (2000个子文件)
TypeExtractor.java 101KB
TypeExtractorTest.java 97KB
MemorySegmentTestBase.java 76KB
MemorySegment.java 73KB
Record.java 73KB
ConfigConstants.java 72KB
HiveDialectITCase.java 66KB
RestClusterClientTest.java 62KB
CopyOnWriteSkipListStateMap.java 58KB
YarnTestBase.java 57KB
HiveDialectQueryITCase.java 57KB
FutureUtils.java 55KB
FileSystem.java 55KB
CliFrontend.java 54KB
HiveTableSourceITCase.java 53KB
HiveServer2EndpointITCase.java 52KB
ExecutionConfig.java 52KB
RestClusterClient.java 51KB
ChangelogKeyedStateBackend.java 48KB
ApplicationDispatcherBootstrapTest.java 48KB
HiveTableSinkITCase.java 46KB
EmbeddedRocksDBStateBackend.java 44KB
RocksDBKeyedStateBackend.java 43KB
CopyOnWriteSkipListStateMapComplexOpTest.java 43KB
TaskManagerOptions.java 43KB
FileInputFormat.java 42KB
JobManagerOptions.java 41KB
LimitedConnectionsFileSystem.java 41KB
RocksDBStateBackendConfigTest.java 40KB
FutureUtilsTest.java 39KB
Configuration.java 38KB
RecordTest.java 38KB
NettyShuffleEnvironmentOptions.java 38KB
OperationsOnFreedSegmentTest.java 37KB
PojoTypeExtractionTest.java 37KB
MemorySegmentUndersizedTest.java 36KB
PojoSerializerUpgradeTestSpecifications.java 36KB
SecurityOptions.java 36KB
HiveRunnerITCase.java 36KB
BasicTypeSerializerUpgradeTestSpecifications.java 36KB
FileInputFormatTest.java 35KB
GenericCsvInputFormatTest.java 34KB
HiveCatalogHiveMetadataTest.java 33KB
YARNSessionCapacitySchedulerITCase.java 33KB
DelimitedInputFormat.java 32KB
SkipListUtils.java 31KB
JarFileCheckerTest.java 30KB
TableEnvHiveConnectorITCase.java 30KB
PojoSerializer.java 30KB
CliFrontendParser.java 30KB
RocksDBMapState.java 29KB
InstantiationUtil.java 29KB
RocksDBNativeMetricOptions.java 29KB
StringValue.java 28KB
FileUtils.java 28KB
RocksDBKeyedStateBackendBuilder.java 28KB
ExceptionUtils.java 28KB
TypeSerializerUpgradeTestBase.java 27KB
RocksDBIncrementalRestoreOperation.java 27KB
PojoSerializerSnapshot.java 27KB
EmbeddedRocksDBStateBackendTest.java 27KB
CollectionExecutor.java 27KB
LimitedConnectionsFileSystemTest.java 27KB
DefaultPackagedProgramRetrieverTest.java 27KB
HiveCatalogITCase.java 27KB
MetricOptions.java 27KB
RocksDBStateBackend.java 26KB
CoreOptions.java 26KB
PackagedProgram.java 26KB
DefaultConfigurableOptionsFactory.java 26KB
HiveDialectAggITCase.java 26KB
HiveCatalogGenericMetadataTest.java 26KB
KryoSerializer.java 26KB
ConfigurationConversionsTest.java 25KB
Transformation.java 25KB
Tuple25.java 25KB
PojoSerializerTest.java 24KB
Tuple24.java 24KB
CompositeTypeSerializerSnapshotTest.java 24KB
HiveLookupJoinITCase.java 24KB
ExpressionKeysTest.java 24KB
FileUtilsTest.java 23KB
Tuple23.java 23KB
RocksDBAsyncSnapshotTest.java 23KB
TupleGenerator.java 23KB
HiveShimV100.java 23KB
ClientTest.java 22KB
SerializerTestBase.java 22KB
ConfigurationUtils.java 22KB
Tuple22.java 22KB
DelimitedInputFormatTest.java 22KB
GenericCsvInputFormat.java 22KB
FileOutputFormatTest.java 22KB
Types.java 22KB
Tuple21.java 21KB
NetUtils.java 21KB
Row.java 21KB
AkkaOptions.java 21KB
ComparatorTestBase.java 21KB
CopyOnWriteSkipListStateMapBasicOpTest.java 21KB
共 2000 条
- 1
- 2
- 3
- 4
- 5
- 6
- 20
资源评论
m0_46190977
- 粉丝: 4
- 资源: 11
上传资源 快速赚钱
- 我的内容管理 展开
- 我的资源 快来上传第一个资源
- 我的收益 登录查看自己的收益
- 我的积分 登录查看自己的积分
- 我的C币 登录后查看C币余额
- 我的收藏
- 我的下载
- 下载帮助
最新资源
资源上传下载、课程学习等过程中有任何疑问或建议,欢迎提出宝贵意见哦~我们会及时处理!
点击此处反馈
安全验证
文档复制为VIP权益,开通VIP直接复制
信息提交成功