Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
@@ -1,18 +1,19 @@
package com.gotocompany.dagger.common.serde.typehandler.repeated;

import com.google.gson.Gson;
import com.google.protobuf.Descriptors;
import com.google.protobuf.DynamicMessage;
import com.gotocompany.dagger.common.core.FieldDescriptorCache;
import com.gotocompany.dagger.common.exceptions.serde.EnumFieldNotFoundException;
import com.gotocompany.dagger.common.serde.parquet.SimpleGroupValidation;
import com.gotocompany.dagger.common.serde.typehandler.TypeHandler;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.common.typeinfo.Types;
import org.apache.flink.api.java.typeutils.ObjectArrayTypeInfo;

import com.google.gson.Gson;
import com.google.protobuf.Descriptors;
import com.google.protobuf.DynamicMessage;
import org.apache.parquet.example.data.simple.SimpleGroup;

import java.util.ArrayList;
import java.util.Arrays;
import java.util.List;
import java.util.stream.Collectors;

Expand Down Expand Up @@ -41,9 +42,28 @@ public boolean canHandle() {

@Override
public DynamicMessage.Builder transformToProtoBuilder(DynamicMessage.Builder builder, Object field) {
if (!canHandle() || field == null) {
return builder;
}
List<Object> rowElements = field.getClass().isArray() ? Arrays.asList((Object[]) field) : (List) field;

List<Descriptors.EnumValueDescriptor> value = rowElements.stream()
.map(this::getEnumValue)
.collect(Collectors.toList());

builder.setField(fieldDescriptor, value);
return builder;
}

private Descriptors.EnumValueDescriptor getEnumValue(Object field) {
String stringValue = String.valueOf(field).trim();
Descriptors.EnumValueDescriptor valueByName = fieldDescriptor.getEnumType().findValueByName(stringValue);
if (valueByName == null) {
throw new EnumFieldNotFoundException("field: " + stringValue + " not found in " + fieldDescriptor.getFullName());
}
return valueByName;
}

@Override
public Object transformFromPostProcessor(Object field) {
return getValue(field);
Expand Down
Original file line number Diff line number Diff line change
@@ -1,22 +1,25 @@
package com.gotocompany.dagger.common.serde.typehandler.repeated;

import com.gotocompany.dagger.common.core.FieldDescriptorCache;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.common.typeinfo.Types;
import org.apache.flink.api.java.typeutils.ObjectArrayTypeInfo;

import com.google.protobuf.Descriptors;
import com.google.protobuf.DynamicMessage;
import com.google.protobuf.InvalidProtocolBufferException;
import com.gotocompany.dagger.common.core.FieldDescriptorCache;
import com.gotocompany.dagger.common.exceptions.serde.EnumFieldNotFoundException;
import com.gotocompany.dagger.consumer.TestBookingLogMessage;
import com.gotocompany.dagger.consumer.TestEnumMessage;
import com.gotocompany.dagger.consumer.TestRepeatedEnumMessage;
import org.apache.flink.api.common.typeinfo.TypeInformation;
import org.apache.flink.api.common.typeinfo.Types;
import org.apache.flink.api.java.typeutils.ObjectArrayTypeInfo;
import org.apache.parquet.example.data.simple.SimpleGroup;
import org.apache.parquet.schema.GroupType;
import org.junit.Assert;
import org.junit.Test;

import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collections;
import java.util.List;

import static org.apache.parquet.schema.PrimitiveType.PrimitiveTypeName.BINARY;
import static org.apache.parquet.schema.PrimitiveType.PrimitiveTypeName.INT64;
Expand All @@ -27,6 +30,7 @@
import static org.junit.Assert.assertTrue;

public class RepeatedEnumHandlerTest {

@Test
public void shouldReturnTrueIfRepeatedEnumFieldDescriptorIsPassed() {
Descriptors.FieldDescriptor repeatedEnumFieldDescriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
Expand All @@ -51,15 +55,6 @@ public void shouldReturnFalseIfFieldDescriptorOtherThanRepeatedEnumTypeIsPassed(
assertFalse(repeatedEnumHandler.canHandle());
}

@Test
public void shouldReturnTheSameBuilderWithoutSettingAnyValue() {
Descriptors.FieldDescriptor repeatedEnumFieldDescriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(repeatedEnumFieldDescriptor);
DynamicMessage.Builder builder = DynamicMessage.newBuilder(repeatedEnumFieldDescriptor.getContainingType());

assertEquals(Collections.EMPTY_LIST, repeatedEnumHandler.transformToProtoBuilder(builder, 123).getField(repeatedEnumFieldDescriptor));
}

@Test
public void shouldReturnTypeInformation() {
Descriptors.FieldDescriptor repeatedEnumFieldDescriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
Expand Down Expand Up @@ -220,4 +215,82 @@ public void shouldTransformValueForParquetAsEmptyStringArrayWhenFieldIsNotInitia

assertArrayEquals(new String[0], actualEnumArray);
}

@Test
public void shouldReturnSameBuilderIfFieldIsDifferentType() {
Descriptors.FieldDescriptor otherFieldDescriptor = TestBookingLogMessage.getDescriptor().findFieldByName("order_number");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestBookingLogMessage.getDescriptor());
RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(otherFieldDescriptor);

repeatedEnumHandler.transformToProtoBuilder(builder, "abc");
assertEquals("", builder.getField(otherFieldDescriptor));
}

@Test
public void shouldReturnSameBuilderIfFieldIsSetAsNull() {
Descriptors.FieldDescriptor descriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestRepeatedEnumMessage.getDescriptor());

RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(descriptor);

repeatedEnumHandler.transformToProtoBuilder(builder, null);
assertEquals(Collections.emptyList(), builder.getField(descriptor));
}

@Test
public void shouldSetValueIfValueIsArray() {
Descriptors.FieldDescriptor descriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestRepeatedEnumMessage.getDescriptor());

RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(descriptor);
String[] a = {"FIRST_ENUM_VALUE"};

repeatedEnumHandler.transformToProtoBuilder(builder, a);
assertEquals(Collections.singletonList(TestEnumMessage.Enum.FIRST_ENUM_VALUE.getValueDescriptor()), builder.getField(descriptor));
}

@Test
public void shouldSetValueIfValueIsList() {
Descriptors.FieldDescriptor descriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestRepeatedEnumMessage.getDescriptor());

RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(descriptor);

repeatedEnumHandler.transformToProtoBuilder(builder, Collections.singletonList("FIRST_ENUM_VALUE"));
assertEquals(Collections.singletonList(TestEnumMessage.Enum.FIRST_ENUM_VALUE.getValueDescriptor()), builder.getField(descriptor));
}

@Test
public void shouldIncludeDataInProtoSerializedFormat() throws InvalidProtocolBufferException {
Descriptors.FieldDescriptor descriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestRepeatedEnumMessage.getDescriptor());

RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(descriptor);

repeatedEnumHandler.transformToProtoBuilder(builder, Arrays.asList("FIRST_ENUM_VALUE", "SECOND_ENUM_VALUE"));
byte[] byteData = builder.build().toByteArray();
DynamicMessage message = DynamicMessage.parseFrom(TestRepeatedEnumMessage.getDescriptor(), byteData);

List<Descriptors.EnumValueDescriptor> expected = Arrays.asList(TestEnumMessage.Enum.FIRST_ENUM_VALUE.getValueDescriptor(), TestEnumMessage.Enum.SECOND_ENUM_VALUE.getValueDescriptor());
assertEquals(expected, message.getField(descriptor));
}

@Test
public void shouldThrowErrorIfEnumValueNotFound() {
Descriptors.FieldDescriptor descriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestRepeatedEnumMessage.getDescriptor());

RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(descriptor);
EnumFieldNotFoundException exception = Assert.assertThrows(EnumFieldNotFoundException.class, () -> repeatedEnumHandler.transformToProtoBuilder(builder, Collections.singletonList("test_enum")));
assertEquals("field: test_enum not found in com.gotocompany.dagger.consumer.TestRepeatedEnumMessage.test_enums", exception.getMessage());
}

@Test
public void shouldThrowErrorIfValueIsNotListOrArray() {
Descriptors.FieldDescriptor descriptor = TestRepeatedEnumMessage.getDescriptor().findFieldByName("test_enums");
DynamicMessage.Builder builder = DynamicMessage.newBuilder(TestRepeatedEnumMessage.getDescriptor());

RepeatedEnumHandler repeatedEnumHandler = new RepeatedEnumHandler(descriptor);
Assert.assertThrows(ClassCastException.class, () -> repeatedEnumHandler.transformToProtoBuilder(builder, "test_enum"));
}
}