You are viewing an old version of this page. View the current version.

Compare with Current View Page History

« Previous Version 55 Next »

Status

Discussion thread-
Vote thread-
JIRA

Unable to render Jira issues macro, execution error.

Release-

Motivation

Flink ETL job consumes data from Source and produces result to Sink. Source creates relationship with Sink through Flink ETL job. Flink needs a mechanism for users to report these relationships to external systems, such as meta system Datahub [1], Atlas [2] and meta store we mentioned in FLIP-276 [3].

This FLIP aims to introduce listener interfaces in Flink, users can implement them to report the meta data and lineage to external systems. The main information is as follows

1. Source and Sink information, such as table name, fields, partition keys, primary keys, watermarks, configurations

2. Job information, such as job id/name, job type, job configuration

3. Relationship between Source/Sink and jobs, such as source and sink and their column lineages.

4. Job execution status changed information, such as job status, exception.

Public Interfaces

CatalogModificationListener

DDL operations such as create/alter/drop tables will generate different events and notify CatalogModificationListener . All events for CatalogModificationListener extend the basic CatalogModificationEvent and listeners can get catalog from it. Some general events for database/table are defined as follows and more events can be implemented based on the requirements in the future.

/**
 * Different events will be fired when a catalog/database/table is modified. The customized listener can get and
 * report specific information from the event according to the event type.
 */
@PublicEvolving
public interface CatalogModificationListener {
    /** The event will be fired when the database/table is modified. */
    void onEvent(CatalogModificationEvent event, CatalogModificationContext context);
}

/* Basic class for catalog modification. */
@PublicEvolving
public abstract class CatalogModificationEvent {
}

/* Context for catalog modification and job lineage events. */
@PublicEvolving
public class CatalogModificationContext {
    /* The name of catalog. */
    String catalogName();

    /* Class of the catalog. */
    Class<? extends Catalog> clazz();

    /* Identifier for the catalog from catalog factory, such as jdbc/iceberg/paimon. */
    Optional<String> factoryIdentifier();

    /* Config for catalog. */
    Map<String, String> config();
}

/* The basic class for database related event. */
public abstract class DatabaseModificationEvent extends CatalogModificationEvent {
    CatalogDatabase database();
}

/* Event for creating database. */
@PublicEvolving
public class CreateDatabaseEvent extends DatabaseModificationEvent {
    boolean ignoreIfExists();
}

/* Event for altering database. */
@PublicEvolving
public class AlterDatabaseEvent extends DatabaseModificationEvent {
    String oldDatabaseName();
    boolean ignoreIfNotExists();
}

/* Event for dropping database. */
@PublicEvolving
public class DropDatabaseEvent extends DatabaseModificationEvent {
    boolean ignoreIfExists();
}

/**
 * Base table event, provides column list, primary keys, partition keys, watermarks and properties in
 * CatalogBaseTable. The table can be source or sink.
 */
public abstract class TableModificationEvent extends CatalogModificationEvent {
    ObjectIdentifier identifier();
    CatalogBaseTable table();
}

/* Event for creating table. */
@PublicEvolving
public class CreateTableEvent extends CatalogModificationEvent {
    boolean ignoreIfExists();
}

/* Event for altering table, provides all changes for old table. */
@PublicEvolving
public class AlterTableEvent extends CatalogModificationEvent {
    List<TableChange> tableChanges();
    boolean ignoreIfExists();
}

/* Event for dropping table. */
@PublicEvolving
public class DropTableEvent extends CatalogModificationEvent {
    boolean ignoreIfExists();
}

/* Factory for catalog modification listener. */
@PublicEvolving
public interface CatalogModificationListenerFactory {
    CatalogModificationListener createListener(Configuration config, ClassLoader classloader);
}

Users may create different catalogs on the same physical catalog, for example, create two hive catalog named hive_catalog1  and hive_catalog2  for the same metastore. The tables hive_catalog1.my_database.my_table  and hive_catalog2.my_database.my_table  are the same table in hive metastore.

In addition, there are two table types: persistent table  and temporal table . The persistent table  can be identified by catalog and database above, while the temporal table  can only be identified by properties in ddl. Different temporal tables with the same connector type and related properties are the same physical table in external system, such as two tables for the same topic in Kafka.

Users can identify the physical connector by CatalogContext and options in CatalogBaseTable through the following steps:

1. Get connector name.

Users can get value of option 'connector' from options in CatalogBaseTable  for temporal tables. If it doesn't exist, users can get factory identifier from CatalogContext as connector name. If none of the above exist, users can define the connector name themselves through Class<? extends Catalog> .

2. Uses can get different properties based on connector name from table options and create connector identifier. Flink has many connectors, and we given the example of kafka options below, users can create kafka identifier with servers, group and topic as needed.

/* Kafka storage identifier options. */
"properties.bootstrap.servers" for Kafka bootstrap servers
"topic" for Kafka Topic
"properties.group.id" for Kafka group id
"topic-pattern" for Kafka topic pattern

For some sensitive information, users can encode and desensitize them in their customized listeners.

JobStatusChangedListener 

Flink creates events and notify JobStatusChangedListener when status of job is changed. There are two types of job status event for the listener: JobCreatedEvent and JobExecutionStatusEvent. JobCreatedEvent will be fired when job is created, it has job lineage and the listener can create lineages for source and sink. JobExecutionStatusEvent has old and new job statuses in runtime and listener can even delete the lineages when job goes to termination.

/**
 * When job is created or its status is changed, Flink will generate job event and notify job status changed listener.
 */
@PublicEvolving
public interface JobStatusChangedListener {
    /* Event will be fired when job status is changed. */
    public void onEvent(JobStatusChangedEvent event);
}

/** Basic job status event. */
@PublicEvolving
public abstract class JobStatusChangedEvent {
    JobID jobId();
    String jobName();
}

/** Job created event with job lineage. */
@PublicEvolving
public class JobCreatedEvent extends JobStatusChangedEvent {
    /* Lineage for the current job. */
    JobLineage lineage();

    /* Job type, TableOrSQL or DataStream. */
    String jobType();

    /* Job execution type, BATCH or STREAMING. */
    String executionType(); 
     
    /* Job configuration. */
    Map<String, String> config();  
}

/** Job status changed event for runtime. */
@PublicEvolving
public class JobExecutionStatusEvent extends JobStatusChangedEvent {
    JobStatus oldStatus();
    JobStatus newStatus();
    @Nullable Throwable exception();
}

/** Factory for job status changed listener. */
@PublicEvolving
public interface JobStatusChangedListenerFactory {
    JobStatusChangedListener createListener(Configuration config, ClassLoader classloader);
}


Job lineage is divided into two layers: the first layer is global abstraction for all Flink jobs and connectors, and the second layer defines the lineages for Table/Sql and DataStream independently based on the first one. 

/**
 * Job lineage is built according to StreamGraph. Users can get sources, sinks and relationships from lineage.
 */
@PublicEvolvig
public class JobLineage {
    /* Source lineage entity list. */
    List<LineageEntity> sources();

    /* Sink lineage entity list. */
    List<LineageEntity> sinks();

    /* Lineage relations from sources to sinks. */
    List<LineageRelation> relations();
}

@PublicEvolving
public class LineageEntity {
    /* Config for the lineage entity. */
    Map<String, String> config();
}

/** Lineage relation from source to sink. */
@PublicEvolving
public class LineageRelation {
    LineageEntity source();
    LineageEntity sink();
}

Job lineage for Table/SQL job

For Table/SQL jobs, Flink creates table lineages according to tables for source and sink. There're column lineages in table lineages, and Flink jobs can create the dependencies between source and sink columns. Flink creates these lineages for Table/SQL jobs from job plan, the entire processing has nothing to do with users.

/** Basic table lineage entity which has catalog context and table in it. */
public class TableLineageEntity extends LineageEntity {
    /* The catalog context of the table lineage entity. */
    public CatalogContext catalogContext();

    /* The table of the table lineage entity. */
    public CatalogBaseTable table();
}

/** Source lineage entity for table. */
@PublicEvolving
public class TableSourceLineageEntity extends TableLineageEntity {
}

/** Sink lineage entity for table. */
@PublicEvolving
public class TableSinkLineageEntity extends TableLineageEntity {
    /* Modify type, INSERT/UPDATE/DELETE. */
    String modifyType();
 
    /* Update mode, APPEND/RETRACT/UPSERT. */
    String updateMode();
    boolean overwrite();
}

/* Table lineage relations from source table to sink table. */
@PublicEvolving
public class TableLineageRelation extends LineageRelation {
    /* Table column lineage relations from source to sink. */
    List<TableColumnLineageRelation> columnRelations();
}

/* Column lineage from source table to sink table. */
@PublicEvolving
public class TableColumnLineageRelation {
    /* Source table column. */
    String sourceColumn();

    /* Sink table column. */
    String sinkColumn();
}

Job lineage for DataStream job

The data structures of connectors in DataStream jobs are much more complex than in Table/SQL jobs, they may be tables, user customized POJO classes or even vector for ML jobs. We added setLineageEntity in DataStreamSource and DataStreamSink which allows users to define job lineage entity using TaleSourceLineageEntity and TableSinkLineageEntity if the connectors are tables, or to implement customized LineageEntity for source and sink based on their specific requirements.

/**
 * Add setLineageEntity method in data stream source.
 */
@Public
public class DataStreamSource {
    private LineageEntity lineageEntity;
 
    public DataStreamSource setLineageEntity(LineageEntity lineageEntity);
}
 
/**
 * Add setLineageEntity and addLineageRelations methods in data stream sink.
 */
@Public
public class DataStreamSink {
    private LineageEntity lineageEntity;
 
    public DataStreamSink setLineageEntity(LineageEntity lineageEntity);

    /* Add lineage relations for data stream jobs. */
    public DataStreamSink addLineageRelations(LineageRelation ... relations);
}

Config Customized Listener

Users should add their listeners to the classpath of client and flink cluster, and config the listener factory in the following options

# Config for catalog modification listeners.
lineage.catalog-modification.listeners: {job catalog listener factory1},{job catalog listener factory2}

# Config for job status changed listeners.
lineage.job-status-changed.listeners: {job status changed listener factory1},{job status changed listener factory2}

Proposed Changes

Use case of job lineage

1. User customized lineage for DataStream

Users can implement customized source and sink lineages for datastream job, for example, KafkaVectorLineageEntity for kafka source and sink as follows.

/** User defined vector source and sink lineage entity. */
public class KafkaVectorLineageEntity extends LineageEntity {
    /* The capacity of source lineage. */
    int capacity();

    /* The value type in the vector. */
    String valueType();
}
 
/* User can use vector source/sink lineages in datastream job. */
Map<String, String> kafkaSourceConf = ...;
Map<String, String> kafkaSinkConf = ...;
StreamExecutionEnvironment env = ...;
KafkaVectorLineageEntity sourceLineageEntity = new KafkaVectorLineageEntity(10, "int", kafkaSourceConf);
KafkaVectorLineageEntity sinkLineageEntity = new KafkaVectorLineageEntity(20, "double", kafkaSinkConf);
env.fromSource(...)
        .setLineageEntity(sourceLineageEntity) // Set source lineage entity
	.map(...).keyBy(..).reduce(..)...
	.sinkTo(...)
        .setLineageEntity(sinkLineageEntity)  // Set sink lineage entity
        .addLineageRelations(                 // Add lineage relations from sources to the current sink
            new LineageRelation(sourceLineageEntity, sinkLineageEntity);
env.execute();

After that, users can cast the source and sink lineage entities to vector lineage entity, get capacity and value type in the customized listeners.

2. Connectors identified by connector identifier

Users may create different tables on a same storage, such as the same Kafka topic. Suppose there's one Kafka topic, two Paimon tables and one Mysql table. Users create these tables and submit three Flink SQL jobs as follows.

a) Create Kafka and Paimon tables for Flink SQL job

-- Create a table my_kafka_table1 for kafka topic 'kafka_topic'
CREATE TABLE my_kafka_table1 (
  val1 STRING,
  val2 STRING,
  val3 STRING
) WITH (
  'connector' = 'kafka',
  'topic' = 'kafka_topic',
  'properties.bootstrap.servers' = 'kafka_bootstrap_server',
  'properties.group.id' = 'kafka_group',
  'format' = 'json'
);

-- Create a Paimon catalog and table for warehouse 'paimon_path1'
CREATE CATALOG paimon_catalog WITH (
    'type'='paimon',
    'warehouse'='paimon_path1'
);
USE CATALOG paimon_catalog;
CREATE TABLE my_paimon_table (
    val1 STRING,
    val2 STRING,
    val3 STRING
) WITH (...);

-- Insert data to Paimon table from Kafka
INSERT INTO my_paimon_table SELECT ... FROM default_catalog.default_database.my_kafka_table1 WHERE ...;

b) Create another Kafka and Paimon tables for Flink SQL job

-- Create another table my_kafka_table2 for kafka topic 'kafka_topic' which is same as my_kafka_table1 above
CREATE TABLE my_kafka_table2 (
  val1 STRING,
  val2 STRING,
  val3 STRING
) WITH (
  'connector' = 'kafka',
  'topic' = 'kafka_topic',
  'properties.bootstrap.servers' = 'kafka_bootstrap_server',
  'properties.group.id' = 'kafka_group',
  'format' = 'json'
);

-- Create a Paimon catalog with the same name 'paimon_catalog' for different warehouse 'paimon_path2'
CREATE CATALOG paimon_catalog WITH (
    'type'='paimon',
    'warehouse'='paimon_path2'
);
USE CATALOG paimon_catalog;
CREATE TABLE my_paimon_table (
    val1 STRING,
    val2 STRING,
    val3 STRING
) WITH (...);

-- Insert data to Paimon table from Kafka
INSERT INTO my_paimon_table SELECT ... FROM default_catalog.default_database.my_kafka_table2 WHERE ...;

c) Create Mysql table for Flink SQL job

-- Create two catalogs for warehouse 'paimon_path1' and 'paimon_path2', there are two different tables 'my_paimon_table'
CREATE CATALOG paimon_catalog1 WITH (
    'type'='paimon',
    'warehouse'='paimon_path1'
);
CREATE CATALOG paimon_catalog2 WITH (
    'type'='paimon',
    'warehouse'='paimon_path2'
);

-- Create mysql table
CREATE TABLE mysql_table (
    val1 STRING,
    val2 STRING,
    val3 STRING
) WITH (
    'connector' = 'jdbc',
    'url' = 'jdbc:mysql://HOST:PORT/my_database',
    'table-name' = 'my_mysql_table',
    'username' = '***',
    'password' = '***'
);

-- Insert data to mysql table from two paimon tales
INSERT INTO mysql_table 
    SELECT ... FROM paimon_catalog1.default.my_paimon_table
        JOIN paimon_catalog2.default.my_paimon_table
    ON ... WHERE ...;


After completing the above operations, we got one Kafka topic, two Paimon tables and one Mysql table which are identified by connector identifier. These tables are associated through Flink jobs, users can report the tables and relationships to datahub[1] as an example which is shown below

Changes for CatalogModificationListener

TableEnvironmentImpl creates customized CatalogModificationListener according to the option lineage.catalog-modification.listeners , and build CatalogManager with the listeners. Some other components such as Sql-Gateway can create CatalogManager with the listeners themselves. Currently all table related operations such as create/alter are in CatalogManager , but database operations are not. We can add database modification operations in CatalogManager  and notify the specified listeners for tables and databases.

/* Listeners and related operations in the catalog manager. */
public final class CatalogManager {
    private final List<CatalogModificationListener> listeners;

    /* Create catalog manager with listener list. */
    private CatalogManager(
            String defaultCatalogName,
            Catalog defaultCatalog,
            DataTypeFactory typeFactory,
            ManagedTableListener managedTableListener,
            List<CatalogModificationListener> listeners);

    /* Notify the listeners with given catalog event. */
    private void notify(CatalogEvent event) {
        listeners.forEach(listener -> listener.onEvent(event));
    }

    /* Notify listener for tables. */
    public void createTable/dropTable/alterTable(...) {
        ....;
        notify(Create Different Table Modification Event);
    }

    /* Add database ddls and notify listener for databases. */
    public void createDatabase/dropDatabase/alterDatabase(...) {
        ....;
        notify(Create Different Database Modification Event); 
    }

    /* Add listeners in Builder for catalog manager. */
    public static final class Builder {
        Builder listeners(List<CatalogModificationListener> listeners);
    }
}

Changes for JobStatusChangedListener

Build JobLogicalPlan in StreamGraph

Flink creates Planner for sql and table jobs which contains exec nodes, then the planner will be converted to Transformation and StreamGraph. DataStream jobs are similar with SQL, Flink creates DataStream and converts it to Transformation and StreamGraph. The job conversion is shown as followed. 

There is a graph structure in StreamGraph , we can create JobLogicalPlan based on StreamGraph easily. 

For table and SQL jobs, Flink translates source and sink exec nodes in planner to Transformation in ExecNodeBase.translateToPlanInternal. There's resolved table in source and sink nodes, Flink can create source and sink lineage based on the table and save them in source/sink transformations.

For DataStream  jobs, DataStreamSource set the source lineage to source transformation in setLineage method, and DataStreamSink does the same thing.

Finally source and sink nodes are translated from transformation and added to StreamGraph in SourceTransformationTranslator.translateInternal  and SinkTransformationTranslator.translateInternal , where their lineages information can be added to StreamGraph too.

Create and notify listeners in RestClusterClient

RestClusterClient  can read option lineage.job-status-changed.listeners from Configuration and create JobStatusChangedListener . RestClusterClient.submitJob is used in AbstractSessionClusterExecutor to submit job, which will convert Pipeline to JobGraph. AbstractSessionClusterExecutor can set StreamGraph in RestClusterClient before calling the submitJob method, then the RestClusterClient can get the StreamGraph and notify the listener before JobGraph is submitted.

/* Set pipeline to client before it submit job graph.  */
public class AbstractSessionClusterExecutor {
    public CompletableFuture<JobClient> execute(
            @Nonnull final Pipeline pipeline,
            @Nonnull final Configuration configuration,
            @Nonnull final ClassLoader userCodeClassloader)
            throws Exception {
        ....

        final ClusterClientProvider<ClusterID> clusterClientProvider =
                    clusterDescriptor.retrieve(clusterID);
        ClusterClient<ClusterID> clusterClient = clusterClientProvider.getClusterClient();

        // Set pipeline to the cluster client before submit job graph
        clusterClient.setPipeline(pipeline); 

        return clusterClient
                    .submitJob(jobGraph)
                    ....  
    }
}

/* Create job submission event and notify listeners before it submit job graph. */
public class RestClusterClient {
    private final List<JobStatusChangedListener> listeners;
    private Pipeline pipeline;

    @Override
    public CompletableFuture<JobID> submitJob(@Nonnull JobGraph jobGraph) {
        // Create event and notify listeners before the job graph is submitted.
        JobSubmissionEvent event = createEventFromPipeline(pipeline);
        if (event != null) {
            listeners.forEach(listener -> listener.onEvent(event));
        }
        
        ....;
    }
}

Job status changed in job manager

JobManager  can create JobStatusChangedListener in DefaultExecutionGraph according to option lineage.job-status-changed.listeners in job configuration. Currently JobManager will call DefaultExecutionGraph.transitionState when the job status is changed, JobStatusChangedListener can be notified in the method as follows.

/* Set pipeline to client before it submit job graph.  */
public class DefaultExecutionGraph {
    private final List<JobStatusChangedListener> executionListeners;

    private boolean transitionState(JobStatus current, JobStatus newState, Throwable error) {
        ....;
        notifyJobStatusHooks(newState, error);
        // notify job execution listeners
        notifyJobStatusChangedListeners(current, newState, error);
        ....;
    }

    private void notifyJobStatusChangedListeners(JobStatus current, JobStatus newState, Throwable error) {
        JobExecutionStatusEvent event = create job execution status event;
        executionListeners.forEach(listener -> listener.onEvent(event));
    }
}

Listener Execution

Multiple listeners are independent, and client/JobManager will notify the listeners synchronously. It is highly recommended NOT to perform any blocking operation inside the listeners. If blocked operations are required, users need to perform asynchronous processing in their customized listeners.

Rejected Alternatives

Report lineage in JobListener

Currently there is JobListener in Flink, users can implement custom listeners and obtain JobClient. We can add lineage information in JobListener and users can report it. However, JobListener is added in FLINK-14992 [4] which is used by Zeppelin and Notebook to manage the job. The lineage information of Flink job is static and we don't need JobClient in the listener. So we reject this proposal and create JobStatusChangedListener for lineage.

Pull job status in JobListener

We can pull the job status in JobListener . As discussed in the previous thread [5], we do not want to do such things in our services too.

Submit job with lineage and Report it in JobMaster

In this FLIP the JobStatusChangedListener will be in Client and JobMaster, which will report lineage information and job status independently. Another proposal is to put the lineage information into JobGraph and then report it in JobMaster . Considering that the lineage information may be relatively large and affect JobGraph , we will report separately in the first phase and then consider merging in the future.

Plan For The Future

  1. We add column lineages in job lineage, but it is not supported in Flink at present. We'd like to implement them in the next FLIP. 
  2. Currently we only supports scan source in this FLIP, lookup join source should be supported later.

  3. Add Job vertex listener for batch mode, such as scheduling and execution status of vertex, execution status of subtask, etc.

  4. Current client will notify listener to report job lineage, it should be reported in JobManager and supports in REST api in the future.


[1] https://datahub.io/

[2] https://atlas.apache.org/#/

[3] FLIP-276: Data Consistency of Streaming and Batch ETL in Flink and Table Store

[4] https://issues.apache.org/jira/browse/FLINK-14992

[5] https://lists.apache.org/thread/ttsc6155bzrtkh8wkkxg2dj689jtvh4t



  • No labels