Skip to content

Commit

Permalink
[FLINK-34549][API] Introduce related components for runtime context
Browse files Browse the repository at this point in the history
  • Loading branch information
reswqa committed May 22, 2024
1 parent 6691dc6 commit c17440c
Show file tree
Hide file tree
Showing 4 changed files with 173 additions and 0 deletions.
Original file line number Diff line number Diff line change
@@ -0,0 +1,42 @@
/*
* 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.datastream.api.context;

import org.apache.flink.annotation.Experimental;

/** {@link JobInfo} contains all the meta information of the job. */
@Experimental
public interface JobInfo {
/**
* Get the name of current job.
*
* @return the name of current job
*/
String getJobName();

/** Get the {@link ExecutionMode} of current job. */
ExecutionMode getExecutionMode();

/** Execution mode of this current job. */
@Experimental
enum ExecutionMode {
STREAMING,
BATCH
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,51 @@
/*
* 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.datastream.api.context;

import org.apache.flink.annotation.Experimental;

/**
* This is responsibility for managing runtime information related to processing time of process
* function.
*/
@Experimental
public interface ProcessingTimeManager {
/**
* Register a processing timer for this process function. `onProcessingTimer` method of this
* function will be invoked as callback if the timer expires.
*
* @param timestamp to trigger timer callback.
*/
void registerTimer(long timestamp);

/**
* Deletes the processing-time timer with the given trigger timestamp. This method has only an
* effect if such a timer was previously registered and did not already expire.
*
* @param timestamp indicates the timestamp of the timer to delete.
*/
void deleteTimer(long timestamp);

/**
* Get the current processing time.
*
* @return current processing time.
*/
long currentTime();
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,34 @@
/*
* 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.datastream.api.context;

import org.apache.flink.annotation.Experimental;

/** This is responsibility for managing runtime information related to state of process function. */
@Experimental
public interface StateManager {
/**
* Get the key of current record.
*
* @return The key of current processed record.
* @throws UnsupportedOperationException if the key can not be extracted for this function, for
* instance, get the key from a non-keyed partition stream.
*/
<K> K getCurrentKey() throws UnsupportedOperationException;
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,46 @@
/*
* 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.datastream.api.context;

import org.apache.flink.annotation.Experimental;

/** {@link TaskInfo} contains all the meta information of the task. */
@Experimental
public interface TaskInfo {
/**
* Get the parallelism of current task.
*
* @return the parallelism of this process function.
*/
int getParallelism();

/**
* Get the max parallelism of current task.
*
* @return The max parallelism.
*/
int getMaxParallelism();

/**
* Get the name of current task.
*
* @return The name of current task.
*/
String getTaskName();
}

0 comments on commit c17440c

Please sign in to comment.