Skip to content

Commit 4028cac

Browse files
HDDS-1347. In OM HA getS3Secret call Should happen only leader OM. (#670)
1 parent 28fb4b5 commit 4028cac

File tree

10 files changed

+333
-4
lines changed

10 files changed

+333
-4
lines changed

hadoop-hdds/common/src/main/java/org/apache/hadoop/ozone/OzoneConsts.java

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -268,6 +268,7 @@ private OzoneConsts() {
268268
public static final String PART_NUMBER_MARKER = "partNumberMarker";
269269
public static final String MAX_PARTS = "maxParts";
270270
public static final String S3_BUCKET = "s3Bucket";
271+
public static final String S3_GETSECRET_USER = "S3GetSecretUser";
271272

272273

273274

hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/audit/OMAction.java

Lines changed: 3 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -69,7 +69,9 @@ public enum OMAction implements AuditAction {
6969
CREATE_DIRECTORY,
7070
CREATE_FILE,
7171
LOOKUP_FILE,
72-
LIST_STATUS;
72+
LIST_STATUS,
73+
74+
GET_S3_SECRET;
7375

7476
@Override
7577
public String getAction() {

hadoop-ozone/common/src/main/java/org/apache/hadoop/ozone/om/exceptions/OMException.java

Lines changed: 4 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -207,7 +207,10 @@ public enum ResultCodes {
207207

208208
RATIS_ERROR, // Error in Ratis server
209209

210-
INVALID_PATH_IN_ACL_REQUEST // Error code when path name is invalid during
210+
INVALID_PATH_IN_ACL_REQUEST, // Error code when path name is invalid during
211211
// acl requests.
212+
213+
USER_MISMATCH // Error code when requested user name passed is different
214+
// from remote user.
212215
}
213216
}

hadoop-ozone/common/src/main/proto/OzoneManagerProtocol.proto

Lines changed: 14 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -156,6 +156,8 @@ message OMRequest {
156156
optional GetAclRequest getAclRequest = 78;
157157

158158
optional PurgeKeysRequest purgeKeysRequest = 81;
159+
160+
optional UpdateGetS3SecretRequest updateGetS3SecretRequest = 82;
159161
}
160162

161163
message OMResponse {
@@ -287,6 +289,9 @@ enum Status {
287289
RATIS_ERROR = 52;
288290

289291
INVALID_PATH_IN_ACL_REQUEST = 53; // Invalid path name in acl request.
292+
293+
USER_MISMATCH = 54; // Error code when requested user name passed is
294+
// different from remote user.
290295
}
291296

292297

@@ -1050,6 +1055,15 @@ message GetS3SecretResponse {
10501055
required S3Secret s3Secret = 2;
10511056
}
10521057

1058+
/**
1059+
This will be used internally by OM to replicate S3 Secret across quorum of
1060+
OM's.
1061+
*/
1062+
message UpdateGetS3SecretRequest {
1063+
required string kerberosID = 1;
1064+
required string awsSecret = 2;
1065+
}
1066+
10531067
/**
10541068
The OM service that takes care of Ozone namespace.
10551069
*/

hadoop-ozone/integration-test/src/test/java/org/apache/hadoop/ozone/TestSecureOzoneCluster.java

Lines changed: 10 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -104,6 +104,7 @@
104104
import static org.junit.Assert.assertFalse;
105105
import static org.junit.Assert.assertNull;
106106
import static org.junit.Assert.assertTrue;
107+
import static org.junit.Assert.fail;
107108
import static org.slf4j.event.Level.INFO;
108109

109110
/**
@@ -689,11 +690,11 @@ public void testGetS3Secret() throws Exception {
689690

690691
//Creates a secret since it does not exist
691692
S3SecretValue firstAttempt = omClient
692-
.getS3Secret("HADOOP/JOHNDOE");
693+
.getS3Secret(UserGroupInformation.getCurrentUser().getUserName());
693694

694695
//Fetches the secret from db since it was created in previous step
695696
S3SecretValue secondAttempt = omClient
696-
.getS3Secret("HADOOP/JOHNDOE");
697+
.getS3Secret(UserGroupInformation.getCurrentUser().getUserName());
697698

698699
//secret fetched on both attempts must be same
699700
assertTrue(firstAttempt.getAwsSecret()
@@ -703,6 +704,13 @@ public void testGetS3Secret() throws Exception {
703704
assertTrue(firstAttempt.getAwsAccessKey()
704705
.equals(secondAttempt.getAwsAccessKey()));
705706

707+
708+
try {
709+
omClient.getS3Secret("HADOOP/JOHNDOE");
710+
fail("testGetS3Secret failed");
711+
} catch (IOException ex) {
712+
GenericTestUtils.assertExceptionContains("USER_MISMATCH", ex);
713+
}
706714
} finally {
707715
if(om != null){
708716
om.stop();

hadoop-ozone/ozone-manager/src/main/java/org/apache/hadoop/ozone/om/OzoneManager.java

Lines changed: 8 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -2683,6 +2683,14 @@ public void deleteS3Bucket(String s3BucketName) throws IOException {
26832683
* {@inheritDoc}
26842684
*/
26852685
public S3SecretValue getS3Secret(String kerberosID) throws IOException{
2686+
UserGroupInformation user = ProtobufRpcEngine.Server.getRemoteUser();
2687+
2688+
// Check whether user name passed is matching with the current user or not.
2689+
if (!user.getUserName().equals(kerberosID)) {
2690+
throw new OMException("User mismatch. Requested user name is " +
2691+
"mismatched " + kerberosID +", with current user " +
2692+
user.getUserName(), OMException.ResultCodes.USER_MISMATCH);
2693+
}
26862694
return s3SecretManager.getS3Secret(kerberosID);
26872695
}
26882696

Original file line numberDiff line numberDiff line change
@@ -0,0 +1,193 @@
1+
/**
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
* <p>
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
* <p>
12+
* Unless required by applicable law or agreed to in writing, software
13+
* distributed under the License is distributed on an "AS IS" BASIS,
14+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15+
* See the License for the specific language governing permissions and
16+
* limitations under the License.
17+
*/
18+
19+
package org.apache.hadoop.ozone.om.request.s3.security;
20+
21+
import java.io.IOException;
22+
import java.util.HashMap;
23+
import java.util.Map;
24+
25+
import com.google.common.base.Optional;
26+
import org.apache.commons.codec.digest.DigestUtils;
27+
import org.slf4j.Logger;
28+
import org.slf4j.LoggerFactory;
29+
30+
import org.apache.hadoop.ipc.ProtobufRpcEngine;
31+
import org.apache.hadoop.ozone.OmUtils;
32+
import org.apache.hadoop.ozone.OzoneConsts;
33+
import org.apache.hadoop.ozone.audit.OMAction;
34+
import org.apache.hadoop.ozone.om.OMMetadataManager;
35+
import org.apache.hadoop.ozone.om.OzoneManager;
36+
import org.apache.hadoop.ozone.om.exceptions.OMException;
37+
import org.apache.hadoop.ozone.om.helpers.S3SecretValue;
38+
import org.apache.hadoop.ozone.om.ratis.utils.OzoneManagerDoubleBufferHelper;
39+
import org.apache.hadoop.ozone.om.request.OMClientRequest;
40+
import org.apache.hadoop.ozone.om.response.OMClientResponse;
41+
import org.apache.hadoop.ozone.om.response.s3.security.S3GetSecretResponse;
42+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos;
43+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.GetS3SecretRequest;
44+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.GetS3SecretResponse;
45+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.OMResponse;
46+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.OMRequest;
47+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.UpdateGetS3SecretRequest;
48+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.S3Secret;
49+
import org.apache.hadoop.security.UserGroupInformation;
50+
import org.apache.hadoop.utils.db.cache.CacheKey;
51+
import org.apache.hadoop.utils.db.cache.CacheValue;
52+
53+
import static org.apache.hadoop.ozone.om.lock.OzoneManagerLock.Resource.S3_SECRET_LOCK;
54+
55+
/**
56+
* Handles GetS3Secret request.
57+
*/
58+
public class S3GetSecretRequest extends OMClientRequest {
59+
60+
private static final Logger LOG =
61+
LoggerFactory.getLogger(S3GetSecretRequest.class);
62+
63+
public S3GetSecretRequest(OMRequest omRequest) {
64+
super(omRequest);
65+
}
66+
67+
@Override
68+
public OMRequest preExecute(OzoneManager ozoneManager) throws IOException {
69+
GetS3SecretRequest s3GetSecretRequest =
70+
getOmRequest().getGetS3SecretRequest();
71+
72+
// Generate S3 Secret to be used by OM quorum.
73+
String kerberosID = s3GetSecretRequest.getKerberosID();
74+
75+
UserGroupInformation user = ProtobufRpcEngine.Server.getRemoteUser();
76+
if (!user.getUserName().equals(kerberosID)) {
77+
throw new OMException("User mismatch. Requested user name is " +
78+
"mismatched " + kerberosID +", with current user " +
79+
user.getUserName(), OMException.ResultCodes.USER_MISMATCH);
80+
}
81+
82+
String s3Secret = DigestUtils.sha256Hex(OmUtils.getSHADigest());
83+
84+
UpdateGetS3SecretRequest updateGetS3SecretRequest =
85+
UpdateGetS3SecretRequest.newBuilder()
86+
.setAwsSecret(s3Secret)
87+
.setKerberosID(kerberosID).build();
88+
89+
// Client issues GetS3Secret request, when received by OM leader
90+
// it will generate s3Secret. Original GetS3Secret request is
91+
// converted to UpdateGetS3Secret request with the generated token
92+
// information. This updated request will be submitted to Ratis. In this
93+
// way S3Secret created by leader, will be replicated across all
94+
// OMs. With this approach, original GetS3Secret request from
95+
// client does not need any proto changes.
96+
OMRequest.Builder omRequest = OMRequest.newBuilder()
97+
.setUserInfo(getUserInfo())
98+
.setUpdateGetS3SecretRequest(updateGetS3SecretRequest)
99+
.setCmdType(getOmRequest().getCmdType())
100+
.setClientId(getOmRequest().getClientId());
101+
102+
if (getOmRequest().hasTraceID()) {
103+
omRequest.setTraceID(getOmRequest().getTraceID());
104+
}
105+
106+
return omRequest.build();
107+
108+
}
109+
110+
@Override
111+
public OMClientResponse validateAndUpdateCache(OzoneManager ozoneManager,
112+
long transactionLogIndex,
113+
OzoneManagerDoubleBufferHelper ozoneManagerDoubleBufferHelper) {
114+
115+
116+
OMClientResponse omClientResponse = null;
117+
OMResponse.Builder omResponse = OMResponse.newBuilder()
118+
.setCmdType(OzoneManagerProtocolProtos.Type.GetS3Secret)
119+
.setStatus(OzoneManagerProtocolProtos.Status.OK)
120+
.setSuccess(true);
121+
boolean acquiredLock = false;
122+
IOException exception = null;
123+
OMMetadataManager omMetadataManager = ozoneManager.getMetadataManager();
124+
UpdateGetS3SecretRequest updateGetS3SecretRequest =
125+
getOmRequest().getUpdateGetS3SecretRequest();
126+
String kerberosID = updateGetS3SecretRequest.getKerberosID();
127+
try {
128+
String awsSecret = updateGetS3SecretRequest.getAwsSecret();
129+
acquiredLock =
130+
omMetadataManager.getLock().acquireLock(S3_SECRET_LOCK, kerberosID);
131+
132+
S3SecretValue s3SecretValue =
133+
omMetadataManager.getS3SecretTable().get(kerberosID);
134+
135+
// If s3Secret for user is not in S3Secret table, add the Secret to cache.
136+
if (s3SecretValue == null) {
137+
omMetadataManager.getS3SecretTable().addCacheEntry(
138+
new CacheKey<>(kerberosID),
139+
new CacheValue<>(Optional.of(new S3SecretValue(kerberosID,
140+
awsSecret)), transactionLogIndex));
141+
} else {
142+
// If it already exists, use the existing one.
143+
awsSecret = s3SecretValue.getAwsSecret();
144+
}
145+
146+
GetS3SecretResponse.Builder getS3SecretResponse = GetS3SecretResponse
147+
.newBuilder().setS3Secret(S3Secret.newBuilder()
148+
.setAwsSecret(awsSecret).setKerberosID(kerberosID));
149+
150+
if (s3SecretValue == null) {
151+
omClientResponse =
152+
new S3GetSecretResponse(new S3SecretValue(kerberosID, awsSecret),
153+
omResponse.setGetS3SecretResponse(getS3SecretResponse).build());
154+
} else {
155+
// As when it already exists, we don't need to add to DB again. So
156+
// set the value to null.
157+
omClientResponse = new S3GetSecretResponse(null,
158+
omResponse.setGetS3SecretResponse(getS3SecretResponse).build());
159+
}
160+
161+
} catch (IOException ex) {
162+
exception = ex;
163+
omClientResponse = new S3GetSecretResponse(null,
164+
createErrorOMResponse(omResponse, ex));
165+
} finally {
166+
if (omClientResponse != null) {
167+
omClientResponse.setFlushFuture(ozoneManagerDoubleBufferHelper.add(
168+
omClientResponse, transactionLogIndex));
169+
}
170+
if (acquiredLock) {
171+
omMetadataManager.getLock().releaseLock(S3_SECRET_LOCK, kerberosID);
172+
}
173+
}
174+
175+
176+
Map<String, String> auditMap = new HashMap<>();
177+
auditMap.put(OzoneConsts.S3_GETSECRET_USER, kerberosID);
178+
179+
// audit log
180+
auditLog(ozoneManager.getAuditLogger(), buildAuditMessage(
181+
OMAction.GET_S3_SECRET, auditMap,
182+
exception, getOmRequest().getUserInfo()));
183+
184+
if (exception == null) {
185+
LOG.debug("Secret for accessKey:{} is generated Successfully",
186+
kerberosID);
187+
} else {
188+
LOG.error("Secret for accessKey:{} is generation failed", kerberosID,
189+
exception);
190+
}
191+
return omClientResponse;
192+
}
193+
}
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,22 @@
1+
/**
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
* <p>
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
* <p>
12+
* Unless required by applicable law or agreed to in writing, software
13+
* distributed under the License is distributed on an "AS IS" BASIS,
14+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15+
* See the License for the specific language governing permissions and
16+
* limitations under the License.
17+
*/
18+
19+
/**
20+
* Package contains classes related to S3 security requests.
21+
*/
22+
package org.apache.hadoop.ozone.om.request.s3.security;
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,56 @@
1+
/**
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
* <p>
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
* <p>
12+
* Unless required by applicable law or agreed to in writing, software
13+
* distributed under the License is distributed on an "AS IS" BASIS,
14+
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15+
* See the License for the specific language governing permissions and
16+
* limitations under the License.
17+
*/
18+
19+
package org.apache.hadoop.ozone.om.response.s3.security;
20+
21+
import org.apache.hadoop.ozone.om.OMMetadataManager;
22+
import org.apache.hadoop.ozone.om.helpers.S3SecretValue;
23+
import org.apache.hadoop.ozone.om.response.OMClientResponse;
24+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos;
25+
import org.apache.hadoop.ozone.protocol.proto.OzoneManagerProtocolProtos.OMResponse;
26+
import org.apache.hadoop.utils.db.BatchOperation;
27+
28+
import javax.annotation.Nonnull;
29+
import javax.annotation.Nullable;
30+
import java.io.IOException;
31+
32+
/**
33+
* Response for GetS3Secret request.
34+
*/
35+
public class S3GetSecretResponse extends OMClientResponse {
36+
37+
38+
private S3SecretValue s3SecretValue;
39+
40+
public S3GetSecretResponse(@Nullable S3SecretValue s3SecretValue,
41+
@Nonnull OMResponse omResponse) {
42+
super(omResponse);
43+
this.s3SecretValue = s3SecretValue;
44+
}
45+
46+
@Override
47+
public void addToDBBatch(OMMetadataManager omMetadataManager,
48+
BatchOperation batchOperation) throws IOException {
49+
50+
if (s3SecretValue != null &&
51+
getOMResponse().getStatus() == OzoneManagerProtocolProtos.Status.OK) {
52+
omMetadataManager.getS3SecretTable().putWithBatch(batchOperation,
53+
s3SecretValue.getKerberosID(), s3SecretValue);
54+
}
55+
}
56+
}

0 commit comments

Comments
 (0)