-
Notifications
You must be signed in to change notification settings - Fork 4.2k
/
PubsubJsonClient.java
361 lines (327 loc) · 13.6 KB
/
PubsubJsonClient.java
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
/*
* 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.beam.sdk.io.gcp.pubsub;
import static org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Preconditions.checkState;
import com.google.api.client.http.HttpRequestInitializer;
import com.google.api.services.pubsub.Pubsub;
import com.google.api.services.pubsub.Pubsub.Projects.Subscriptions;
import com.google.api.services.pubsub.Pubsub.Projects.Topics;
import com.google.api.services.pubsub.model.AcknowledgeRequest;
import com.google.api.services.pubsub.model.ListSubscriptionsResponse;
import com.google.api.services.pubsub.model.ListTopicsResponse;
import com.google.api.services.pubsub.model.ModifyAckDeadlineRequest;
import com.google.api.services.pubsub.model.PublishRequest;
import com.google.api.services.pubsub.model.PublishResponse;
import com.google.api.services.pubsub.model.PubsubMessage;
import com.google.api.services.pubsub.model.PullRequest;
import com.google.api.services.pubsub.model.PullResponse;
import com.google.api.services.pubsub.model.ReceivedMessage;
import com.google.api.services.pubsub.model.Subscription;
import com.google.api.services.pubsub.model.Topic;
import com.google.auth.Credentials;
import com.google.auth.http.HttpCredentialsAdapter;
import com.google.cloud.hadoop.util.ChainingHttpRequestInitializer;
import com.google.protobuf.ByteString;
import java.io.IOException;
import java.util.ArrayList;
import java.util.HashMap;
import java.util.List;
import java.util.Map;
import java.util.TreeMap;
import org.apache.beam.sdk.extensions.gcp.util.RetryHttpRequestInitializer;
import org.apache.beam.sdk.extensions.gcp.util.Transport;
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.annotations.VisibleForTesting;
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.MoreObjects;
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.base.Strings;
import org.apache.beam.vendor.guava.v26_0_jre.com.google.common.collect.ImmutableList;
import org.checkerframework.checker.nullness.qual.Nullable;
/** A Pubsub client using JSON transport. */
@SuppressWarnings({
"nullness" // TODO(https://github.com/apache/beam/issues/20497)
})
public class PubsubJsonClient extends PubsubClient {
private static class PubsubJsonClientFactory implements PubsubClientFactory {
private static HttpRequestInitializer chainHttpRequestInitializer(
Credentials credential, HttpRequestInitializer httpRequestInitializer) {
if (credential == null) {
return httpRequestInitializer;
} else {
return new ChainingHttpRequestInitializer(
new HttpCredentialsAdapter(credential), httpRequestInitializer);
}
}
@Override
public PubsubClient newClient(
@Nullable String timestampAttribute, @Nullable String idAttribute, PubsubOptions options)
throws IOException {
return newClient(timestampAttribute, idAttribute, options, null);
}
@Override
public PubsubClient newClient(
@Nullable String timestampAttribute,
@Nullable String idAttribute,
PubsubOptions options,
String rootUrlOverride)
throws IOException {
Pubsub pubsub =
new Pubsub.Builder(
Transport.getTransport(),
Transport.getJsonFactory(),
chainHttpRequestInitializer(
options.getGcpCredential(),
// Do not log 404. It clutters the output and is possibly even required by the
// caller.
new RetryHttpRequestInitializer(ImmutableList.of(404))))
.setRootUrl(MoreObjects.firstNonNull(rootUrlOverride, options.getPubsubRootUrl()))
.setApplicationName(options.getAppName())
.setGoogleClientRequestInitializer(options.getGoogleApiTrace())
.build();
return new PubsubJsonClient(timestampAttribute, idAttribute, pubsub);
}
@Override
public String getKind() {
return "Json";
}
}
/** Factory for creating Pubsub clients using Json transport. */
public static final PubsubClientFactory FACTORY = new PubsubJsonClientFactory();
/**
* Attribute to use for custom timestamps, or {@literal null} if should use Pubsub publish time
* instead.
*/
private final @Nullable String timestampAttribute;
/** Attribute to use for custom ids, or {@literal null} if should use Pubsub provided ids. */
private final @Nullable String idAttribute;
/** Underlying JSON transport. */
private Pubsub pubsub;
@VisibleForTesting
PubsubJsonClient(
@Nullable String timestampAttribute, @Nullable String idAttribute, Pubsub pubsub) {
this.timestampAttribute = timestampAttribute;
this.idAttribute = idAttribute;
this.pubsub = pubsub;
}
@Override
public void close() {
// Nothing to close.
}
@Override
public int publish(TopicPath topic, List<OutgoingMessage> outgoingMessages) throws IOException {
List<PubsubMessage> pubsubMessages = new ArrayList<>(outgoingMessages.size());
for (OutgoingMessage outgoingMessage : outgoingMessages) {
PubsubMessage pubsubMessage =
new PubsubMessage().encodeData(outgoingMessage.message().getData().toByteArray());
pubsubMessage.setAttributes(getMessageAttributes(outgoingMessage));
if (!outgoingMessage.message().getOrderingKey().isEmpty()) {
pubsubMessage.setOrderingKey(outgoingMessage.message().getOrderingKey());
}
// N.B. publishTime and messageId are intentionally not set on the message that is published
pubsubMessages.add(pubsubMessage);
}
PublishRequest request = new PublishRequest().setMessages(pubsubMessages);
PublishResponse response =
pubsub.projects().topics().publish(topic.getPath(), request).execute();
return response.getMessageIds().size();
}
private Map<String, String> getMessageAttributes(OutgoingMessage outgoingMessage) {
Map<String, String> attributes = null;
if (outgoingMessage.message().getAttributesMap() == null) {
attributes = new TreeMap<>();
} else {
attributes = new TreeMap<>(outgoingMessage.message().getAttributesMap());
}
if (timestampAttribute != null) {
attributes.put(timestampAttribute, String.valueOf(outgoingMessage.timestampMsSinceEpoch()));
}
if (idAttribute != null && !Strings.isNullOrEmpty(outgoingMessage.recordId())) {
attributes.put(idAttribute, outgoingMessage.recordId());
}
return attributes;
}
@Override
@SuppressWarnings("ProtoFieldNullComparison")
public List<IncomingMessage> pull(
long requestTimeMsSinceEpoch,
SubscriptionPath subscription,
int batchSize,
boolean returnImmediately)
throws IOException {
PullRequest request =
new PullRequest().setReturnImmediately(returnImmediately).setMaxMessages(batchSize);
PullResponse response =
pubsub.projects().subscriptions().pull(subscription.getPath(), request).execute();
if (response.getReceivedMessages() == null || response.getReceivedMessages().isEmpty()) {
return ImmutableList.of();
}
List<IncomingMessage> incomingMessages = new ArrayList<>(response.getReceivedMessages().size());
for (ReceivedMessage message : response.getReceivedMessages()) {
PubsubMessage pubsubMessage = message.getMessage();
Map<String, String> attributes;
if (pubsubMessage.getAttributes() != null) {
attributes = pubsubMessage.getAttributes();
} else {
attributes = new HashMap<>();
}
// Payload.
byte[] elementBytes = pubsubMessage.getData() == null ? null : pubsubMessage.decodeData();
if (elementBytes == null) {
elementBytes = new byte[0];
}
// Timestamp.
long timestampMsSinceEpoch;
if (Strings.isNullOrEmpty(timestampAttribute)) {
timestampMsSinceEpoch = parseTimestampAsMsSinceEpoch(message.getMessage().getPublishTime());
} else {
timestampMsSinceEpoch = extractTimestampAttribute(timestampAttribute, attributes);
}
// Ack id.
String ackId = message.getAckId();
checkState(!Strings.isNullOrEmpty(ackId));
// Record id, if any.
@Nullable String recordId = null;
if (idAttribute != null) {
recordId = attributes.get(idAttribute);
}
if (Strings.isNullOrEmpty(recordId)) {
// Fall back to the Pubsub provided message id.
recordId = pubsubMessage.getMessageId();
}
com.google.pubsub.v1.PubsubMessage.Builder protoMessage =
com.google.pubsub.v1.PubsubMessage.newBuilder();
protoMessage.setData(ByteString.copyFrom(elementBytes));
protoMessage.putAllAttributes(attributes);
// PubsubMessage uses `null` to represent no ordering key where we want a default of "".
if (pubsubMessage.getOrderingKey() != null) {
protoMessage.setOrderingKey(pubsubMessage.getOrderingKey());
} else {
protoMessage.setOrderingKey("");
}
incomingMessages.add(
IncomingMessage.of(
protoMessage.build(),
timestampMsSinceEpoch,
requestTimeMsSinceEpoch,
ackId,
recordId));
}
return incomingMessages;
}
@Override
public void acknowledge(SubscriptionPath subscription, List<String> ackIds) throws IOException {
AcknowledgeRequest request = new AcknowledgeRequest().setAckIds(ackIds);
pubsub
.projects()
.subscriptions()
.acknowledge(subscription.getPath(), request)
.execute(); // ignore Empty result.
}
@Override
public void modifyAckDeadline(
SubscriptionPath subscription, List<String> ackIds, int deadlineSeconds) throws IOException {
ModifyAckDeadlineRequest request =
new ModifyAckDeadlineRequest().setAckIds(ackIds).setAckDeadlineSeconds(deadlineSeconds);
pubsub
.projects()
.subscriptions()
.modifyAckDeadline(subscription.getPath(), request)
.execute(); // ignore Empty result.
}
@Override
public void createTopic(TopicPath topic) throws IOException {
pubsub
.projects()
.topics()
.create(topic.getPath(), new Topic())
.execute(); // ignore Topic result.
}
@Override
public void deleteTopic(TopicPath topic) throws IOException {
pubsub.projects().topics().delete(topic.getPath()).execute(); // ignore Empty result.
}
@Override
public List<TopicPath> listTopics(ProjectPath project) throws IOException {
Topics.List request = pubsub.projects().topics().list(project.getPath());
ListTopicsResponse response = request.execute();
if (response.getTopics() == null || response.getTopics().isEmpty()) {
return ImmutableList.of();
}
List<TopicPath> topics = new ArrayList<>(response.getTopics().size());
while (true) {
for (Topic topic : response.getTopics()) {
topics.add(topicPathFromPath(topic.getName()));
}
if (Strings.isNullOrEmpty(response.getNextPageToken())) {
break;
}
request.setPageToken(response.getNextPageToken());
response = request.execute();
}
return topics;
}
@Override
public void createSubscription(
TopicPath topic, SubscriptionPath subscription, int ackDeadlineSeconds) throws IOException {
Subscription request =
new Subscription().setTopic(topic.getPath()).setAckDeadlineSeconds(ackDeadlineSeconds);
pubsub
.projects()
.subscriptions()
.create(subscription.getPath(), request)
.execute(); // ignore Subscription result.
}
@Override
public void deleteSubscription(SubscriptionPath subscription) throws IOException {
pubsub
.projects()
.subscriptions()
.delete(subscription.getPath())
.execute(); // ignore Empty result.
}
@Override
public List<SubscriptionPath> listSubscriptions(ProjectPath project, TopicPath topic)
throws IOException {
Subscriptions.List request = pubsub.projects().subscriptions().list(project.getPath());
ListSubscriptionsResponse response = request.execute();
if (response.getSubscriptions() == null || response.getSubscriptions().isEmpty()) {
return ImmutableList.of();
}
List<SubscriptionPath> subscriptions = new ArrayList<>(response.getSubscriptions().size());
while (true) {
for (Subscription subscription : response.getSubscriptions()) {
if (subscription.getTopic().equals(topic.getPath())) {
subscriptions.add(subscriptionPathFromPath(subscription.getName()));
}
}
if (Strings.isNullOrEmpty(response.getNextPageToken())) {
break;
}
request.setPageToken(response.getNextPageToken());
response = request.execute();
}
return subscriptions;
}
@Override
public int ackDeadlineSeconds(SubscriptionPath subscription) throws IOException {
Subscription response = pubsub.projects().subscriptions().get(subscription.getPath()).execute();
return response.getAckDeadlineSeconds();
}
@Override
public boolean isEOF() {
return false;
}
}