Skip to content

Commit

Permalink
[AMORO-2683] Modify the job name of SparkOptimizer and FlinkOptimizer (
Browse files Browse the repository at this point in the history
…apache#2690)

* [AMORO-2683] Modify the job name of SparkOptimizer and FlinkOptimizer

* fix comment

* spotless apply

---------

Co-authored-by: huyuanfeng <[email protected]>
  • Loading branch information
huyuanfeng2018 and huyuanfeng authored Mar 31, 2024
1 parent 45db986 commit c17e8ca
Show file tree
Hide file tree
Showing 2 changed files with 8 additions and 5 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -36,7 +36,7 @@
public class FlinkOptimizer extends Optimizer {
private static final Logger LOG = LoggerFactory.getLogger(FlinkOptimizer.class);

private static final String JOB_NAME = "amoro-flink-optimizer";
private static final String JOB_NAME_FORMAT = "amoro-flink-optimizer-%s";

public FlinkOptimizer(OptimizerConfig config) {
super(config, () -> new OptimizerToucher(config), (i) -> new FlinkOptimizerExecutor(config, i));
Expand Down Expand Up @@ -64,7 +64,7 @@ public static void main(String[] args) throws CmdLineException {
.setParallelism(1);

try {
env.execute(JOB_NAME);
env.execute(String.format(JOB_NAME_FORMAT, optimizerConfig.getResourceId()));
} catch (Exception e) {
LOG.error("Execute flink optimizer failed", e);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -31,7 +31,7 @@
/** The {@code SparkOptimizer} acts as an entrypoint of the spark program */
public class SparkOptimizer extends Optimizer {
private static final Logger LOG = LoggerFactory.getLogger(SparkOptimizer.class);
private static final String APP_NAME = "amoro-spark-optimizer";
private static final String APP_NAME_FORMAT = "amoro-spark-optimizer-%s";

public SparkOptimizer(OptimizerConfig config, JavaSparkContext jsc) {
super(
Expand All @@ -41,9 +41,12 @@ public SparkOptimizer(OptimizerConfig config, JavaSparkContext jsc) {
}

public static void main(String[] args) throws Exception {
SparkSession spark = SparkSession.builder().appName(APP_NAME).getOrCreate();
JavaSparkContext jsc = new JavaSparkContext(spark.sparkContext());
OptimizerConfig config = new OptimizerConfig(args);
SparkSession spark =
SparkSession.builder()
.appName(String.format(APP_NAME_FORMAT, config.getResourceId()))
.getOrCreate();
JavaSparkContext jsc = new JavaSparkContext(spark.sparkContext());
if (!jsc.getConf().getBoolean("spark.dynamicAllocation.enabled", false)) {
LOG.warn(
"To better utilize computing resources, it is recommended to enable 'spark.dynamicAllocation.enabled' "
Expand Down

0 comments on commit c17e8ca

Please sign in to comment.