Skip to content

Commit b8d4811

Browse files
author
toutian
committed
Merge branch '1.8_krb_add' into 'v1.8.0_dev'
kerberos 属性配置 See merge request !177
2 parents a926ddd + 68af37b commit b8d4811

File tree

3 files changed

+17
-38
lines changed

3 files changed

+17
-38
lines changed

README.md

Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -50,10 +50,11 @@
5050
* 操作系统:理论上不限
5151
* kerberos环境需要在flink-conf.yaml配置security.kerberos.login.keytab以及security.kerberos.login.principal参数,配置案例:
5252
```
53+
## hadoop配置文件路径
5354
fs.hdfs.hadoopconf: /Users/maqi/tmp/hadoopconf/hadoop_250
5455
security.kerberos.login.use-ticket-cache: true
55-
security.kerberos.login.keytab: /Users/maqi/tmp/hadoopconf/hadoop_250/yanxi.keytab
56-
security.kerberos.login.principal: yanxi@DTSTACK.COM
56+
security.kerberos.login.keytab: /Users/maqi/tmp/hadoopconf/hadoop_250/maqi.keytab
57+
security.kerberos.login.principal: maqi@DTSTACK.COM
5758
security.kerberos.login.contexts: Client,KafkaClient
5859
zookeeper.sasl.service-name: zookeeper
5960
zookeeper.sasl.login-context-name: Client

launcher/src/main/java/com/dtstack/flink/sql/launcher/perjob/PerJobClusterClientBuilder.java

Lines changed: 12 additions & 33 deletions
Original file line numberDiff line numberDiff line change
@@ -26,6 +26,8 @@
2626
import org.apache.flink.configuration.Configuration;
2727
import com.google.common.base.Strings;
2828
import org.apache.flink.runtime.jobgraph.JobGraph;
29+
import org.apache.flink.runtime.security.SecurityConfiguration;
30+
import org.apache.flink.runtime.security.SecurityUtils;
2931
import org.apache.flink.yarn.AbstractYarnClusterDescriptor;
3032
import org.apache.flink.yarn.YarnClusterDescriptor;
3133
import org.apache.hadoop.fs.Path;
@@ -55,45 +57,40 @@ public class PerJobClusterClientBuilder {
5557

5658
private static final Logger LOG = LoggerFactory.getLogger(PerJobClusterClientBuilder.class);
5759

58-
private static String KEYTAB = "security.kerberos.login.keytab";
59-
60-
private static String PRINCIPAL = "security.kerberos.login.principal";
60+
private static final String DEFAULT_CONF_DIR = "./";
6161

6262
private YarnClient yarnClient;
6363

6464
private YarnConfiguration yarnConf;
6565

66-
public void init(String yarnConfDir, Properties conf) throws IOException {
66+
private Configuration flinkConfig;
67+
68+
public void init(String yarnConfDir, Configuration flinkConfig, Properties userConf) throws Exception {
6769

6870
if(Strings.isNullOrEmpty(yarnConfDir)) {
6971
throw new RuntimeException("parameters of yarn is required");
7072
}
73+
userConf.forEach((key, val) -> flinkConfig.setString(key.toString(), val.toString()));
74+
this.flinkConfig = flinkConfig;
75+
SecurityUtils.install(new SecurityConfiguration(flinkConfig));
7176

7277
yarnConf = YarnConfLoader.getYarnConf(yarnConfDir);
73-
74-
if (isKerberos(conf)){
75-
String keytab = (String) conf.get(KEYTAB);
76-
String principal = (String) conf.get(PRINCIPAL);
77-
login(yarnConf, keytab, principal);
78-
}
79-
8078
yarnClient = YarnClient.createYarnClient();
8179
yarnClient.init(yarnConf);
8280
yarnClient.start();
8381

8482
System.out.println("----init yarn success ----");
8583
}
8684

87-
public AbstractYarnClusterDescriptor createPerJobClusterDescriptor(Properties confProp, String flinkJarPath, Options launcherOptions, JobGraph jobGraph, Configuration flinkConfig)
85+
public AbstractYarnClusterDescriptor createPerJobClusterDescriptor(String flinkJarPath, Options launcherOptions, JobGraph jobGraph)
8886
throws MalformedURLException {
8987

90-
confProp.forEach((key, val) -> flinkConfig.setString(key.toString(), val.toString()));
91-
String flinkConf = StringUtils.isEmpty(launcherOptions.getFlinkconf()) ? "" : launcherOptions.getFlinkconf();
88+
String flinkConf = StringUtils.isEmpty(launcherOptions.getFlinkconf()) ? DEFAULT_CONF_DIR : launcherOptions.getFlinkconf();
9289
AbstractYarnClusterDescriptor clusterDescriptor = getClusterDescriptor(flinkConfig, yarnConf, flinkConf);
9390

9491
if (StringUtils.isNotBlank(flinkJarPath)) {
9592
if (!new File(flinkJarPath).exists()) {
96-
throw new RuntimeException("The Flink jar path is not exist");
93+
throw new RuntimeException("The param '-flinkJarPath' ref dir is not exist");
9794
}
9895
}
9996

@@ -163,22 +160,4 @@ private AbstractYarnClusterDescriptor getClusterDescriptor(
163160
false);
164161
}
165162

166-
private boolean isKerberos(Properties conf){
167-
String keytab = (String) conf.get(KEYTAB);
168-
if (StringUtils.isNotBlank(keytab)){
169-
return true;
170-
} else {
171-
return false;
172-
}
173-
}
174-
175-
private void login(org.apache.hadoop.conf.Configuration conf, String keytab, String principal) throws IOException {
176-
if (StringUtils.isEmpty(principal)){
177-
throw new RuntimeException(PRINCIPAL + " must not be null!");
178-
}
179-
UserGroupInformation.setConfiguration(conf);
180-
UserGroupInformation.loginUserFromKeytab(principal, keytab);
181-
LOG.info("login successfully! keytab: " + keytab + "principal: " + principal);
182-
LOG.info("UGI: " + UserGroupInformation.getCurrentUser());
183-
}
184163
}

launcher/src/main/java/com/dtstack/flink/sql/launcher/perjob/PerJobSubmitter.java

Lines changed: 2 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -62,11 +62,10 @@ public static String submit(Options launcherOptions, JobGraph jobGraph, Configur
6262
ClusterSpecification clusterSpecification = FLinkPerJobResourceUtil.createClusterSpecification(confProperties);
6363

6464
PerJobClusterClientBuilder perJobClusterClientBuilder = new PerJobClusterClientBuilder();
65-
perJobClusterClientBuilder.init(launcherOptions.getYarnconf(), confProperties);
65+
perJobClusterClientBuilder.init(launcherOptions.getYarnconf(), flinkConfig, confProperties);
6666

6767
String flinkJarPath = launcherOptions.getFlinkJarPath();
68-
69-
AbstractYarnClusterDescriptor yarnClusterDescriptor = perJobClusterClientBuilder.createPerJobClusterDescriptor(confProperties, flinkJarPath, launcherOptions, jobGraph, flinkConfig);
68+
AbstractYarnClusterDescriptor yarnClusterDescriptor = perJobClusterClientBuilder.createPerJobClusterDescriptor(flinkJarPath, launcherOptions, jobGraph);
7069
ClusterClient<ApplicationId> clusterClient = yarnClusterDescriptor.deployJobCluster(clusterSpecification, jobGraph,true);
7170

7271
String applicationId = clusterClient.getClusterId().toString();

0 commit comments

Comments
 (0)